kirktrue commented on code in PR #14640:
URL: https://github.com/apache/kafka/pull/14640#discussion_r1421165461


##########
clients/src/test/java/org/apache/kafka/clients/consumer/internals/MembershipManagerImplTest.java:
##########
@@ -790,6 +812,197 @@ public void 
testOnSubscriptionUpdatedTransitionsToJoiningOnlyIfNotInGroup() {
         verify(membershipManager, never()).transitionToJoining();
     }
 
+    @Test
+    public void testListenerCallbacksBasic() {
+        // Step 1: set up mocks
+        MembershipManagerImpl membershipManager = createMemberInStableState();
+        mockOwnedPartition("topic1", 0);
+        CounterConsumerRebalanceListener listener = new 
CounterConsumerRebalanceListener();
+        ConsumerRebalanceListenerInvoker invoker = 
consumerRebalanceListenerInvoker();
+
+        
when(subscriptionState.rebalanceListener()).thenReturn(Optional.of(listener));
+        doNothing().when(subscriptionState).markPendingRevocation(anySet());
+
+        // Step 2: put the state machine into the appropriate... state
+        receiveEmptyAssignment(membershipManager);
+        assertEquals(MemberState.RECONCILING, membershipManager.state());
+        assertEquals(Collections.emptySet(), 
membershipManager.currentAssignment());
+        assertTrue(membershipManager.reconciliationInProgress());
+        listener.assertCounts(0, 0, 0);
+
+        // Step 2: revoke partitions
+        performCallback(
+                membershipManager,
+                invoker,
+                ConsumerRebalanceListenerMethodName.ON_PARTITIONS_REVOKED,
+                topicPartitions(new TopicPartition("topic1", 0))
+        );
+
+        // Step 3: assign partitions
+        performCallback(
+                membershipManager,
+                invoker,
+                ConsumerRebalanceListenerMethodName.ON_PARTITIONS_ASSIGNED,
+                Collections.emptySortedSet()
+        );
+
+        // Step 4: Receive ack and make sure we're done and our listener was 
called appropriately
+        membershipManager.onHeartbeatRequestSent();
+        assertEquals(MemberState.STABLE, membershipManager.state());
+
+        listener.assertCounts(1, 1, 0);
+    }
+
+    @Test
+    public void testListenerCallbacksNoListeners() {
+        // Step 1: set up mocks
+        MembershipManagerImpl membershipManager = createMemberInStableState();
+        mockOwnedPartition("topic1", 0);
+        CounterConsumerRebalanceListener listener = new 
CounterConsumerRebalanceListener();
+
+        
when(subscriptionState.rebalanceListener()).thenReturn(Optional.empty());
+        doNothing().when(subscriptionState).markPendingRevocation(anySet());
+
+        // Step 2: put the state machine into the appropriate... state
+        receiveEmptyAssignment(membershipManager);
+        assertEquals(MemberState.ACKNOWLEDGING, membershipManager.state());
+        assertEquals(Collections.emptySet(), 
membershipManager.currentAssignment());
+        assertFalse(membershipManager.reconciliationInProgress());
+        assertEquals(0, backgroundEventQueue.size());
+        listener.assertCounts(0, 0, 0);
+
+        // Step 3: Receive ack and make sure we're done and our listener was 
called appropriately
+        membershipManager.onHeartbeatRequestSent();
+        assertEquals(MemberState.STABLE, membershipManager.state());
+
+        listener.assertCounts(0, 0, 0);
+    }
+
+    @Test
+    public void testOnPartitionsLostNoError() {
+        mockOwnedPartition("topic1", 0);
+        testOnPartitionsLost(Optional.empty());
+    }
+
+    @Test
+    public void testOnPartitionsLostError() {
+        mockOwnedPartition("topic1", 0);
+        testOnPartitionsLost(Optional.of(new KafkaException("Intentional error 
for test")));
+    }
+
+    private void testOnPartitionsLost(Optional<RuntimeException> lostError) {
+        // Step 1: set up mocks
+        MembershipManagerImpl membershipManager = createMemberInStableState();
+        CounterConsumerRebalanceListener listener = new 
CounterConsumerRebalanceListener(
+                Optional.empty(),
+                Optional.empty(),
+                lostError
+        );
+        ConsumerRebalanceListenerInvoker invoker = 
consumerRebalanceListenerInvoker();
+
+        
when(subscriptionState.rebalanceListener()).thenReturn(Optional.of(listener));
+        doNothing().when(subscriptionState).markPendingRevocation(anySet());
+
+        // Step 2: put the state machine into the appropriate... state
+        membershipManager.transitionToFenced();
+        assertEquals(MemberState.FENCED, membershipManager.state());
+        assertEquals(Collections.emptySet(), 
membershipManager.currentAssignment());
+        listener.assertCounts(0, 0, 0);
+
+        // Step 3: invoke the callback
+        performCallback(
+                membershipManager,
+                invoker,
+                ConsumerRebalanceListenerMethodName.ON_PARTITIONS_LOST,
+                topicPartitions(new TopicPartition("topic1", 0))
+        );
+
+        // Step 4: Receive ack and make sure we're done and our listener was 
called appropriately
+        membershipManager.onHeartbeatRequestSent();
+        assertEquals(MemberState.JOINING, membershipManager.state());
+
+        listener.assertCounts(0, 0, 1);
+    }
+
+    private ConsumerRebalanceListenerInvoker 
consumerRebalanceListenerInvoker() {
+        ConsumerCoordinatorMetrics coordinatorMetrics = new 
ConsumerCoordinatorMetrics(
+                subscriptionState,
+                new Metrics(),
+                "test-");
+        return new ConsumerRebalanceListenerInvoker(
+                new LogContext(),
+                subscriptionState,
+                new MockTime(1),
+                coordinatorMetrics
+        );
+    }
+
+    private SortedSet<TopicPartition> topicPartitions(TopicPartition... 
topicPartitions) {
+        SortedSet<TopicPartition> revokedPartitions = new TreeSet<>(new 
Utils.TopicPartitionComparator());
+        revokedPartitions.addAll(Arrays.asList(topicPartitions));
+        return revokedPartitions;
+    }
+
+    private void performCallback(MembershipManagerImpl membershipManager,
+                                 ConsumerRebalanceListenerInvoker invoker,
+                                 ConsumerRebalanceListenerMethodName 
methodName,
+                                 SortedSet<TopicPartition> partitions) {
+        // Set up our mock application event handler & background event 
processor.
+        ApplicationEventHandler applicationEventHandler = 
mock(ApplicationEventHandler.class);
+
+        doAnswer(a -> {
+            ConsumerRebalanceListenerCallbackCompletedEvent completedEvent = 
a.getArgument(0);
+            
membershipManager.consumerRebalanceListenerCallbackCompleted(completedEvent);
+            return null;
+        
}).when(applicationEventHandler).add(any(ConsumerRebalanceListenerCallbackCompletedEvent.class));
+
+        // We expect only our enqueued event in the background queue.
+        assertEquals(1, backgroundEventQueue.size());
+        assertNotNull(backgroundEventQueue.peek());
+        assertInstanceOf(ConsumerRebalanceListenerCallbackNeededEvent.class, 
backgroundEventQueue.peek());
+        ConsumerRebalanceListenerCallbackNeededEvent neededEvent = 
(ConsumerRebalanceListenerCallbackNeededEvent) backgroundEventQueue.poll();
+        assertNotNull(neededEvent);
+        assertEquals(methodName, neededEvent.methodName());
+        assertEquals(partitions, neededEvent.partitions());
+
+        final Exception e;
+
+        switch (methodName) {
+            case ON_PARTITIONS_REVOKED:
+                e = invoker.invokePartitionsRevoked(partitions);
+                break;
+
+            case ON_PARTITIONS_ASSIGNED:
+                e = invoker.invokePartitionsAssigned(partitions);
+                break;
+
+            case ON_PARTITIONS_LOST:
+                e = invoker.invokePartitionsLost(partitions);
+                break;
+
+            default:
+                throw new IllegalArgumentException("The method " + methodName 
+ " to invoke was not expected");
+        }
+
+        final Optional<KafkaException> error;
+
+        if (e != null) {
+            if (e instanceof KafkaException)
+                error = Optional.of((KafkaException) e);
+            else
+                error = Optional.of(new KafkaException("User rebalance 
callback throws an error", e));
+        } else {
+            error = Optional.empty();
+        }

Review Comment:
   Yes, I agree. There's more ceremony than I would like 😞
   
   The `ConsumerRebalanceListenerInvoker` is shared code between both the 
`LegacyKafkaConsumer` and `AsyncKafkaConsumer` implementations. It allows the 
callback execution, logging, metrics updates, and exception handling to be 
consistent between them.
   
   The `ApplicationEventHandler` is the mechanism that allows the application 
thread to communicate with the background network I/O thread. It's message 
passing between two threads 🤷‍♂️ While the `ConsumerRebalanceListener` 
callbacks are to be executed on the application thread, the rest of the 
reconciliation logic lies on the background thread.



##########
clients/src/test/java/org/apache/kafka/clients/consumer/internals/MembershipManagerImplTest.java:
##########
@@ -790,6 +812,197 @@ public void 
testOnSubscriptionUpdatedTransitionsToJoiningOnlyIfNotInGroup() {
         verify(membershipManager, never()).transitionToJoining();
     }
 
+    @Test
+    public void testListenerCallbacksBasic() {
+        // Step 1: set up mocks
+        MembershipManagerImpl membershipManager = createMemberInStableState();
+        mockOwnedPartition("topic1", 0);
+        CounterConsumerRebalanceListener listener = new 
CounterConsumerRebalanceListener();
+        ConsumerRebalanceListenerInvoker invoker = 
consumerRebalanceListenerInvoker();
+
+        
when(subscriptionState.rebalanceListener()).thenReturn(Optional.of(listener));
+        doNothing().when(subscriptionState).markPendingRevocation(anySet());
+
+        // Step 2: put the state machine into the appropriate... state
+        receiveEmptyAssignment(membershipManager);
+        assertEquals(MemberState.RECONCILING, membershipManager.state());
+        assertEquals(Collections.emptySet(), 
membershipManager.currentAssignment());
+        assertTrue(membershipManager.reconciliationInProgress());
+        listener.assertCounts(0, 0, 0);
+
+        // Step 2: revoke partitions
+        performCallback(
+                membershipManager,
+                invoker,
+                ConsumerRebalanceListenerMethodName.ON_PARTITIONS_REVOKED,
+                topicPartitions(new TopicPartition("topic1", 0))
+        );
+
+        // Step 3: assign partitions
+        performCallback(
+                membershipManager,
+                invoker,
+                ConsumerRebalanceListenerMethodName.ON_PARTITIONS_ASSIGNED,
+                Collections.emptySortedSet()
+        );
+
+        // Step 4: Receive ack and make sure we're done and our listener was 
called appropriately
+        membershipManager.onHeartbeatRequestSent();
+        assertEquals(MemberState.STABLE, membershipManager.state());
+
+        listener.assertCounts(1, 1, 0);
+    }
+
+    @Test
+    public void testListenerCallbacksNoListeners() {
+        // Step 1: set up mocks
+        MembershipManagerImpl membershipManager = createMemberInStableState();
+        mockOwnedPartition("topic1", 0);
+        CounterConsumerRebalanceListener listener = new 
CounterConsumerRebalanceListener();
+
+        
when(subscriptionState.rebalanceListener()).thenReturn(Optional.empty());
+        doNothing().when(subscriptionState).markPendingRevocation(anySet());
+
+        // Step 2: put the state machine into the appropriate... state
+        receiveEmptyAssignment(membershipManager);
+        assertEquals(MemberState.ACKNOWLEDGING, membershipManager.state());
+        assertEquals(Collections.emptySet(), 
membershipManager.currentAssignment());
+        assertFalse(membershipManager.reconciliationInProgress());
+        assertEquals(0, backgroundEventQueue.size());
+        listener.assertCounts(0, 0, 0);
+
+        // Step 3: Receive ack and make sure we're done and our listener was 
called appropriately
+        membershipManager.onHeartbeatRequestSent();
+        assertEquals(MemberState.STABLE, membershipManager.state());
+
+        listener.assertCounts(0, 0, 0);
+    }
+
+    @Test
+    public void testOnPartitionsLostNoError() {
+        mockOwnedPartition("topic1", 0);
+        testOnPartitionsLost(Optional.empty());
+    }
+
+    @Test
+    public void testOnPartitionsLostError() {
+        mockOwnedPartition("topic1", 0);
+        testOnPartitionsLost(Optional.of(new KafkaException("Intentional error 
for test")));
+    }
+
+    private void testOnPartitionsLost(Optional<RuntimeException> lostError) {
+        // Step 1: set up mocks
+        MembershipManagerImpl membershipManager = createMemberInStableState();
+        CounterConsumerRebalanceListener listener = new 
CounterConsumerRebalanceListener(
+                Optional.empty(),
+                Optional.empty(),
+                lostError
+        );
+        ConsumerRebalanceListenerInvoker invoker = 
consumerRebalanceListenerInvoker();
+
+        
when(subscriptionState.rebalanceListener()).thenReturn(Optional.of(listener));
+        doNothing().when(subscriptionState).markPendingRevocation(anySet());
+
+        // Step 2: put the state machine into the appropriate... state
+        membershipManager.transitionToFenced();
+        assertEquals(MemberState.FENCED, membershipManager.state());
+        assertEquals(Collections.emptySet(), 
membershipManager.currentAssignment());
+        listener.assertCounts(0, 0, 0);
+
+        // Step 3: invoke the callback
+        performCallback(
+                membershipManager,
+                invoker,
+                ConsumerRebalanceListenerMethodName.ON_PARTITIONS_LOST,
+                topicPartitions(new TopicPartition("topic1", 0))
+        );
+
+        // Step 4: Receive ack and make sure we're done and our listener was 
called appropriately
+        membershipManager.onHeartbeatRequestSent();
+        assertEquals(MemberState.JOINING, membershipManager.state());
+
+        listener.assertCounts(0, 0, 1);
+    }
+
+    private ConsumerRebalanceListenerInvoker 
consumerRebalanceListenerInvoker() {
+        ConsumerCoordinatorMetrics coordinatorMetrics = new 
ConsumerCoordinatorMetrics(
+                subscriptionState,
+                new Metrics(),
+                "test-");
+        return new ConsumerRebalanceListenerInvoker(
+                new LogContext(),
+                subscriptionState,
+                new MockTime(1),
+                coordinatorMetrics
+        );
+    }
+
+    private SortedSet<TopicPartition> topicPartitions(TopicPartition... 
topicPartitions) {
+        SortedSet<TopicPartition> revokedPartitions = new TreeSet<>(new 
Utils.TopicPartitionComparator());
+        revokedPartitions.addAll(Arrays.asList(topicPartitions));
+        return revokedPartitions;
+    }
+
+    private void performCallback(MembershipManagerImpl membershipManager,
+                                 ConsumerRebalanceListenerInvoker invoker,
+                                 ConsumerRebalanceListenerMethodName 
methodName,
+                                 SortedSet<TopicPartition> partitions) {
+        // Set up our mock application event handler & background event 
processor.
+        ApplicationEventHandler applicationEventHandler = 
mock(ApplicationEventHandler.class);
+
+        doAnswer(a -> {
+            ConsumerRebalanceListenerCallbackCompletedEvent completedEvent = 
a.getArgument(0);
+            
membershipManager.consumerRebalanceListenerCallbackCompleted(completedEvent);
+            return null;
+        
}).when(applicationEventHandler).add(any(ConsumerRebalanceListenerCallbackCompletedEvent.class));
+
+        // We expect only our enqueued event in the background queue.
+        assertEquals(1, backgroundEventQueue.size());
+        assertNotNull(backgroundEventQueue.peek());
+        assertInstanceOf(ConsumerRebalanceListenerCallbackNeededEvent.class, 
backgroundEventQueue.peek());
+        ConsumerRebalanceListenerCallbackNeededEvent neededEvent = 
(ConsumerRebalanceListenerCallbackNeededEvent) backgroundEventQueue.poll();
+        assertNotNull(neededEvent);
+        assertEquals(methodName, neededEvent.methodName());
+        assertEquals(partitions, neededEvent.partitions());
+
+        final Exception e;
+
+        switch (methodName) {
+            case ON_PARTITIONS_REVOKED:
+                e = invoker.invokePartitionsRevoked(partitions);
+                break;
+
+            case ON_PARTITIONS_ASSIGNED:
+                e = invoker.invokePartitionsAssigned(partitions);
+                break;
+
+            case ON_PARTITIONS_LOST:
+                e = invoker.invokePartitionsLost(partitions);
+                break;
+
+            default:
+                throw new IllegalArgumentException("The method " + methodName 
+ " to invoke was not expected");
+        }
+
+        final Optional<KafkaException> error;
+
+        if (e != null) {
+            if (e instanceof KafkaException)
+                error = Optional.of((KafkaException) e);
+            else
+                error = Optional.of(new KafkaException("User rebalance 
callback throws an error", e));
+        } else {
+            error = Optional.empty();
+        }

Review Comment:
   Yes, I agree. There's more ceremony than I would like 😞
   
   The `ConsumerRebalanceListenerInvoker` is shared code between both the 
`LegacyKafkaConsumer` and `AsyncKafkaConsumer` implementations. It allows the 
callback execution, logging, metrics updates, and exception handling to be 
consistent between them.
   
   The `ApplicationEventHandler` is the mechanism that allows the application 
thread to communicate with the background network I/O thread. It's message 
passing between two threads 🤷‍♂️ While the `ConsumerRebalanceListener` 
callbacks are to be executed on the application thread, the rest of the 
reconciliation logic lies on the background thread.



-- 
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: jira-unsubscr...@kafka.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org

Reply via email to