Skip to content

Commit 0815d70

Browse files
authored
KAFKA-18160 Interrupting or waking up onPartitionsAssigned in AsyncConsumer can cause the ConsumerRebalanceListenerCallbackCompletedEvent to be skipped (#18089)
Reviewers: Chia-Ping Tsai <[email protected]>
1 parent fef625c commit 0815d70

File tree

6 files changed

+240
-97
lines changed

6 files changed

+240
-97
lines changed

clients/src/main/java/org/apache/kafka/clients/consumer/internals/AbstractMembershipManager.java

+8-2
Original file line numberDiff line numberDiff line change
@@ -1175,10 +1175,16 @@ private CompletableFuture<Void> assignPartitions(
11751175

11761176
// Invoke user call back.
11771177
CompletableFuture<Void> result = signalPartitionsAssigned(addedPartitions);
1178+
// Enable newly added partitions to start fetching and updating positions for them.
11781179
result.whenComplete((__, exception) -> {
11791180
if (exception == null) {
1180-
// Enable newly added partitions to start fetching and updating positions for them.
1181-
subscriptions.enablePartitionsAwaitingCallback(addedPartitions);
1181+
// Enable assigned partitions to start fetching and updating positions for them.
1182+
// We use assignedPartitions here instead of addedPartitions because there's a chance that the callback
1183+
// might throw an exception, leaving addedPartitions empty. This would result in the poll operation
1184+
// returning no records, as no topic partitions are marked as fetchable. In contrast, with the classic consumer,
1185+
// if the first callback fails but the next one succeeds, polling can still retrieve data. To align with
1186+
// this behavior, we rely on assignedPartitions to avoid such scenarios.
1187+
subscriptions.enablePartitionsAwaitingCallback(toTopicPartitionSet(assignedPartitions));
11821188
} else {
11831189
// Keeping newly added partitions as non-fetchable after the callback failure.
11841190
// They will be retried on the next reconciliation loop, until it succeeds or the

clients/src/main/java/org/apache/kafka/clients/consumer/internals/AsyncKafkaConsumer.java

+18-13
Original file line numberDiff line numberDiff line change
@@ -84,6 +84,7 @@
8484
import org.apache.kafka.common.errors.InterruptException;
8585
import org.apache.kafka.common.errors.InvalidGroupIdException;
8686
import org.apache.kafka.common.errors.TimeoutException;
87+
import org.apache.kafka.common.errors.WakeupException;
8788
import org.apache.kafka.common.internals.ClusterResourceListeners;
8889
import org.apache.kafka.common.metrics.KafkaMetric;
8990
import org.apache.kafka.common.metrics.Metrics;
@@ -2072,23 +2073,27 @@ static ConsumerRebalanceListenerCallbackCompletedEvent invokeRebalanceCallbacks(
20722073
ConsumerRebalanceListenerMethodName methodName,
20732074
SortedSet<TopicPartition> partitions,
20742075
CompletableFuture<Void> future) {
2075-
final Exception e;
2076+
Exception e;
20762077

2077-
switch (methodName) {
2078-
case ON_PARTITIONS_REVOKED:
2079-
e = rebalanceListenerInvoker.invokePartitionsRevoked(partitions);
2080-
break;
2078+
try {
2079+
switch (methodName) {
2080+
case ON_PARTITIONS_REVOKED:
2081+
e = rebalanceListenerInvoker.invokePartitionsRevoked(partitions);
2082+
break;
20812083

2082-
case ON_PARTITIONS_ASSIGNED:
2083-
e = rebalanceListenerInvoker.invokePartitionsAssigned(partitions);
2084-
break;
2084+
case ON_PARTITIONS_ASSIGNED:
2085+
e = rebalanceListenerInvoker.invokePartitionsAssigned(partitions);
2086+
break;
20852087

2086-
case ON_PARTITIONS_LOST:
2087-
e = rebalanceListenerInvoker.invokePartitionsLost(partitions);
2088-
break;
2088+
case ON_PARTITIONS_LOST:
2089+
e = rebalanceListenerInvoker.invokePartitionsLost(partitions);
2090+
break;
20892091

2090-
default:
2091-
throw new IllegalArgumentException("The method " + methodName.fullyQualifiedMethodName() + " to invoke was not expected");
2092+
default:
2093+
throw new IllegalArgumentException("The method " + methodName.fullyQualifiedMethodName() + " to invoke was not expected");
2094+
}
2095+
} catch (WakeupException | InterruptException ex) {
2096+
e = ex;
20922097
}
20932098

20942099
final Optional<KafkaException> error;

clients/src/main/java/org/apache/kafka/clients/consumer/internals/SubscriptionState.java

+2-2
Original file line numberDiff line numberDiff line change
@@ -898,8 +898,8 @@ public synchronized void assignFromSubscribedAwaitingCallback(Collection<TopicPa
898898
}
899899

900900
/**
901-
* Enable fetching and updating positions for the given partitions that were added to the
902-
* assignment, but waiting for the onPartitionsAssigned callback to complete. This is
901+
* Enable fetching and updating positions for the given partitions that were assigned to the
902+
* consumer, but waiting for the onPartitionsAssigned callback to complete. This is
903903
* expected to be used by the async consumer.
904904
*/
905905
public synchronized void enablePartitionsAwaitingCallback(Collection<TopicPartition> partitions) {

clients/src/test/java/org/apache/kafka/clients/consumer/internals/ConsumerMembershipManagerTest.java

+24-7
Original file line numberDiff line numberDiff line change
@@ -29,6 +29,8 @@
2929
import org.apache.kafka.common.TopicIdPartition;
3030
import org.apache.kafka.common.TopicPartition;
3131
import org.apache.kafka.common.Uuid;
32+
import org.apache.kafka.common.errors.InterruptException;
33+
import org.apache.kafka.common.errors.WakeupException;
3234
import org.apache.kafka.common.message.ConsumerGroupHeartbeatResponseData;
3335
import org.apache.kafka.common.message.ConsumerGroupHeartbeatResponseData.Assignment;
3436
import org.apache.kafka.common.message.ConsumerGroupHeartbeatResponseData.TopicPartitions;
@@ -96,6 +98,7 @@
9698
import static org.mockito.Mockito.verify;
9799
import static org.mockito.Mockito.when;
98100

101+
@SuppressWarnings("ClassDataAbstractionCoupling")
99102
public class ConsumerMembershipManagerTest {
100103

101104
private static final String GROUP_ID = "test-group";
@@ -1738,14 +1741,20 @@ public void testListenerCallbacksBasic() {
17381741

17391742
@Test
17401743
public void testListenerCallbacksThrowsErrorOnPartitionsRevoked() {
1744+
testErrorsOnPartitionsRevoked(new WakeupException());
1745+
testErrorsOnPartitionsRevoked(new InterruptException("Intentional onPartitionsRevoked() error"));
1746+
testErrorsOnPartitionsRevoked(new IllegalArgumentException("Intentional onPartitionsRevoked() error"));
1747+
}
1748+
1749+
private void testErrorsOnPartitionsRevoked(RuntimeException error) {
17411750
// Step 1: set up mocks
17421751
String topicName = "topic1";
17431752
Uuid topicId = Uuid.randomUuid();
17441753

17451754
ConsumerMembershipManager membershipManager = createMemberInStableState();
17461755
mockOwnedPartition(membershipManager, topicId, topicName);
17471756
CounterConsumerRebalanceListener listener = new CounterConsumerRebalanceListener(
1748-
Optional.of(new IllegalArgumentException("Intentional onPartitionsRevoked() error")),
1757+
Optional.ofNullable(error),
17491758
Optional.empty(),
17501759
Optional.empty()
17511760
);
@@ -1792,14 +1801,20 @@ public void testListenerCallbacksThrowsErrorOnPartitionsRevoked() {
17921801

17931802
@Test
17941803
public void testListenerCallbacksThrowsErrorOnPartitionsAssigned() {
1804+
testErrorsOnPartitionsAssigned(new WakeupException());
1805+
testErrorsOnPartitionsAssigned(new InterruptException("Intentional error"));
1806+
testErrorsOnPartitionsAssigned(new IllegalArgumentException("Intentional error"));
1807+
}
1808+
1809+
private void testErrorsOnPartitionsAssigned(RuntimeException error) {
17951810
// Step 1: set up mocks
17961811
ConsumerMembershipManager membershipManager = createMemberInStableState();
17971812
String topicName = "topic1";
17981813
Uuid topicId = Uuid.randomUuid();
17991814
mockOwnedPartition(membershipManager, topicId, topicName);
18001815
CounterConsumerRebalanceListener listener = new CounterConsumerRebalanceListener(
18011816
Optional.empty(),
1802-
Optional.of(new IllegalArgumentException("Intentional onPartitionsAssigned() error")),
1817+
Optional.ofNullable(error),
18031818
Optional.empty()
18041819
);
18051820
ConsumerRebalanceListenerInvoker invoker = consumerRebalanceListenerInvoker();
@@ -1879,7 +1894,7 @@ public void testAddedPartitionsTemporarilyDisabledAwaitingOnPartitionsAssignedCa
18791894
true
18801895
);
18811896

1882-
verify(subscriptionState).enablePartitionsAwaitingCallback(addedPartitions);
1897+
verify(subscriptionState).enablePartitionsAwaitingCallback(assignedPartitions);
18831898
}
18841899

18851900
@Test
@@ -1915,12 +1930,14 @@ public void testAddedPartitionsNotEnabledAfterFailedOnPartitionsAssignedCallback
19151930

19161931
@Test
19171932
public void testOnPartitionsLostNoError() {
1918-
testOnPartitionsLost(Optional.empty());
1933+
testOnPartitionsLost(null);
19191934
}
19201935

19211936
@Test
19221937
public void testOnPartitionsLostError() {
1923-
testOnPartitionsLost(Optional.of(new KafkaException("Intentional error for test")));
1938+
testOnPartitionsLost(new KafkaException("Intentional error for test"));
1939+
testOnPartitionsLost(new WakeupException());
1940+
testOnPartitionsLost(new InterruptException("Intentional error for test"));
19241941
}
19251942

19261943
private void assertLeaveGroupDueToExpiredPollAndTransitionToStale(ConsumerMembershipManager membershipManager) {
@@ -2054,7 +2071,7 @@ private void mockPartitionOwnedAndNewPartitionAdded(String topicName,
20542071
receiveAssignment(topicId, Arrays.asList(partitionOwned, partitionAdded), membershipManager);
20552072
}
20562073

2057-
private void testOnPartitionsLost(Optional<RuntimeException> lostError) {
2074+
private void testOnPartitionsLost(RuntimeException lostError) {
20582075
// Step 1: set up mocks
20592076
ConsumerMembershipManager membershipManager = createMemberInStableState();
20602077
String topicName = "topic1";
@@ -2063,7 +2080,7 @@ private void testOnPartitionsLost(Optional<RuntimeException> lostError) {
20632080
CounterConsumerRebalanceListener listener = new CounterConsumerRebalanceListener(
20642081
Optional.empty(),
20652082
Optional.empty(),
2066-
lostError
2083+
Optional.ofNullable(lostError)
20672084
);
20682085
ConsumerRebalanceListenerInvoker invoker = consumerRebalanceListenerInvoker();
20692086

core/src/test/java/kafka/clients/consumer/AsyncKafkaConsumerIntegrationTest.java

-73
This file was deleted.

0 commit comments

Comments
 (0)