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


##########
clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchRequestManager.java:
##########
@@ -65,16 +67,59 @@ protected void maybeThrowAuthFailure(Node node) {
         networkClientDelegate.maybeThrowAuthFailure(node);
     }
 
+    /**
+     * Request that a fetch request be issued to the cluster to pull down the 
next batch of records.
+     *
+     * <p/>
+     *
+     * The returned {@link CompletableFuture} is {@link 
CompletableFuture#complete(Object) completed} when the
+     * fetch request(s) have been created and enqueued into the network 
client's outgoing send buffer.
+     * It is <em>not completed</em> when the network client has received the 
data.
+     *
+     * @return Future for which the caller can wait to ensure that the 
requests have been enqueued
+     */
+    public CompletableFuture<Void> requestFetch() {
+        CompletableFuture<Void> future = new CompletableFuture<>();
+
+        if (pendingFetchRequestFuture != null) {
+            // In this case, we have an outstanding fetch request, so chain 
the newly created future to be
+            // invoked when the outstanding fetch request is completed.
+            pendingFetchRequestFuture.whenComplete((value, exception) -> {
+                if (exception != null) {

Review Comment:
   Just to make sure I understand: are you stating that because the 
_pendingFetchRequestFuture_ is only ever `complete()`-ed and not 
`completeExceptionally()`-ed, that the code in `requestFetch()` could be 
simplified to:
   
   ```java
   if (pendingFetchRequestFuture != null) {
       // In this case, we have an outstanding fetch request, so chain the 
newly created
       // future to be completed when the outstanding fetch request is 
completed.
       pendingFetchRequestFuture.whenComplete((__, ___) -> 
future.complete(null));
   } else {
       pendingFetchRequestFuture = future;
   }
   ```
   
   Or am I misunderstanding?
   
   Thanks!



-- 
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: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to