snuyanzin commented on code in PR #25767: URL: https://github.com/apache/flink/pull/25767#discussion_r1876453346
########## flink-runtime/src/test/java/org/apache/flink/runtime/leaderelection/DefaultLeaderElectionServiceTest.java: ########## @@ -1246,6 +1269,73 @@ private void testNonBlockingCall( testInstance.close(); } + /** + * This test is used to verify FLINK-36451 where we observed concurrent nested locks being + * acquired from the {@link LeaderContender} and from the {@link DefaultLeaderElectionService}. + */ + @Test + void testNestedDeadlockInLeadershipConfirmation() throws Exception { + final AtomicReference<LeaderInformationRegister> leaderInformationStorage = + new AtomicReference<>(LeaderInformationRegister.empty()); + try (final DefaultLeaderElectionService testInstance = + new DefaultLeaderElectionService( + TestingLeaderElectionDriver.newBuilder( + new AtomicBoolean(false), + leaderInformationStorage, + new AtomicBoolean(false)) + ::build)) { + final String componentId = "test-component"; + final LeaderElection leaderElection = testInstance.createLeaderElection(componentId); + + // we need the lock to be acquired once for the leadership grant and once for the + // revocation + final CountDownLatch contenderLockAcquireLatch = new CountDownLatch(2); + final OneShotLatch grantReceivedLatch = new OneShotLatch(); + + final AtomicBoolean contenderLeadership = new AtomicBoolean(false); + final TestingGenericLeaderContender leaderContender = + TestingGenericLeaderContender.newBuilder() + .setPreLockAcquireAction(contenderLockAcquireLatch::countDown) + .setGrantLeadershipConsumer( + ignoredSessionId -> { + contenderLeadership.set(true); + grantReceivedLatch.trigger(); + }) + .setRevokeLeadershipRunnable(() -> contenderLeadership.set(false)) + .build(); + + leaderElection.startLeaderElection(leaderContender); + + final UUID leaderSessionId = UUID.randomUUID(); + testInstance.onGrantLeadership(leaderSessionId); + grantReceivedLatch.await(); + + final CompletableFuture<Void> revocationFuture; + final CompletableFuture<Void> confirmLeadershipFuture; + synchronized (leaderContender.getLock()) { + revocationFuture = CompletableFuture.runAsync(testInstance::onRevokeLeadership); + contenderLockAcquireLatch.await(); + confirmLeadershipFuture = + leaderElection.confirmLeadershipAsync(leaderSessionId, "random-address"); + } + + assertThatFuture(revocationFuture).eventuallySucceeds(); + assertThatFuture(confirmLeadershipFuture).eventuallySucceeds(); + + assertThat(contenderLeadership).isFalse(); + assertThat(leaderInformationStorage.get().forComponentId(componentId).isPresent()) + .as( + "The LeaderInformation is empty because the leadership confirmation succeeded the " + + "leadership revocation which resulted in no leader information being written out to " + + "the HA backend.") + .isFalse(); Review Comment: ```suggestion assertThat(leaderInformationStorage.get().forComponentId(componentId)) .as( "The LeaderInformation is empty because the leadership confirmation succeeded the " + "leadership revocation which resulted in no leader information being written out to " + "the HA backend.") .isEmpty(); ``` since anyway need to change it, how about this i guess it should work for both java 8 and 11 -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org