junrao commented on code in PR #14123:
URL: https://github.com/apache/kafka/pull/14123#discussion_r1281188661


##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/ApplicationEventProcessor.java:
##########
@@ -106,12 +106,18 @@ private boolean process(final CommitApplicationEvent 
event) {
     private boolean process(final OffsetFetchApplicationEvent event) {
         Optional<RequestManager> commitRequestManger = 
registry.get(RequestManager.Type.COMMIT);
         if (!commitRequestManger.isPresent()) {
-            event.future.completeExceptionally(new KafkaException("Unable to 
fetch committed offset because the " +
+            event.future().completeExceptionally(new KafkaException("Unable to 
fetch committed offset because the " +

Review Comment:
   Would it be better to call `event.future()` once and reuse the result?



##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/events/OffsetFetchApplicationEvent.java:
##########
@@ -19,25 +19,33 @@
 import org.apache.kafka.clients.consumer.OffsetAndMetadata;
 import org.apache.kafka.common.TopicPartition;
 
-import java.time.Duration;
 import java.util.Map;
 import java.util.Set;
 import java.util.concurrent.CompletableFuture;
-import java.util.concurrent.ExecutionException;
-import java.util.concurrent.TimeUnit;
-import java.util.concurrent.TimeoutException;
 
 public class OffsetFetchApplicationEvent extends ApplicationEvent {
-    final public CompletableFuture<Map<TopicPartition, OffsetAndMetadata>> 
future;
-    public final Set<TopicPartition> partitions;
+    private final CompletableFuture<Map<TopicPartition, OffsetAndMetadata>> 
future;
+    private final Set<TopicPartition> partitions;
 
     public OffsetFetchApplicationEvent(final Set<TopicPartition> partitions) {
         super(Type.FETCH_COMMITTED_OFFSET);
         this.partitions = partitions;
         this.future = new CompletableFuture<>();
     }
 
-    public Map<TopicPartition, OffsetAndMetadata> complete(final Duration 
duration) throws ExecutionException, InterruptedException, TimeoutException {
-        return future.get(duration.toMillis(), TimeUnit.MILLISECONDS);
+    public CompletableFuture<Map<TopicPartition, OffsetAndMetadata>> future() {
+        return this.future;

Review Comment:
   this.future => future



##########
clients/src/test/java/org/apache/kafka/clients/consumer/internals/PrototypeAsyncConsumerTest.java:
##########
@@ -150,15 +151,33 @@ public void testCommitAsync_UserSuppliedCallback() {
     }
 
     @Test
-    @SuppressWarnings("unchecked")
     public void testCommitted() {
         Set<TopicPartition> mockTopicPartitions = 
mockTopicPartitionOffset().keySet();
-        mockConstruction(OffsetFetchApplicationEvent.class, (mock, ctx) -> {
-            when(mock.complete(any())).thenReturn(new HashMap<>());
-        });
-        consumer = newConsumer(time, new StringDeserializer(), new 
StringDeserializer());
-        assertDoesNotThrow(() -> consumer.committed(mockTopicPartitions, 
Duration.ofMillis(1)));
-        
verify(eventHandler).add(ArgumentMatchers.isA(OffsetFetchApplicationEvent.class));
+        CompletableFuture<Map<TopicPartition, OffsetAndMetadata>> 
committedFuture = new CompletableFuture<>();
+        try (MockedConstruction<OffsetFetchApplicationEvent> mockConstruction =

Review Comment:
   `mockConstruction` is unused. Also, why do we wrap under `try`? Do we expect 
`mockConstruction` to be closed in the end?



##########
core/src/test/scala/integration/kafka/api/BaseAsyncConsumerTest.scala:
##########
@@ -35,8 +39,12 @@ class BaseAsyncConsumerTest extends AbstractConsumerTest {
     consumer.commitAsync(cb)
     waitUntilTrue(() => {
       cb.successCount == 1
-    }, "wait until commit is completed successfully", 5000)
+    }, "wait until commit is completed successfully", 
defaultBlockingAPITimeoutMs)
+    val committedOffset = consumer.committed(Set(tp).asJava, 
Duration.ofMillis(defaultBlockingAPITimeoutMs))
+
     assertTrue(consumer.assignment.contains(tp))
+    assertNotNull(committedOffset)
+    assertNull(committedOffset.get(tp))

Review Comment:
   Hmm, why would the committed offset be null for tp?



##########
core/src/test/scala/integration/kafka/api/BaseAsyncConsumerTest.scala:
##########
@@ -17,12 +17,16 @@
 package kafka.api
 
 import kafka.utils.TestUtils.waitUntilTrue
-import org.junit.jupiter.api.Assertions.assertTrue
+import org.junit.jupiter.api.Assertions.{assertNotNull, assertNull, assertTrue}
 import org.junit.jupiter.api.Test
 
+import java.time.Duration
 import scala.jdk.CollectionConverters._
 
+

Review Comment:
   extra line



-- 
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