kirktrue commented on code in PR #14406: URL: https://github.com/apache/kafka/pull/14406#discussion_r1351011656
########## clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchRequestManager.java: ########## @@ -0,0 +1,181 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.apache.kafka.clients.consumer.internals; + +import java.util.ArrayList; +import java.util.LinkedList; +import java.util.List; +import java.util.Map; +import java.util.Queue; +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.Future; +import java.util.function.BiConsumer; +import java.util.function.Predicate; +import java.util.stream.Collectors; + +import org.apache.kafka.clients.ClientRequest; +import org.apache.kafka.clients.ClientResponse; +import org.apache.kafka.clients.FetchSessionHandler; +import org.apache.kafka.clients.NetworkClient; +import org.apache.kafka.clients.consumer.internals.NetworkClientDelegate.PollResult; +import org.apache.kafka.clients.consumer.internals.NetworkClientDelegate.UnsentRequest; +import org.apache.kafka.common.Node; +import org.apache.kafka.common.requests.FetchRequest; +import org.apache.kafka.common.utils.LogContext; +import org.apache.kafka.common.utils.Time; + +/** + * {@code FetchRequestManager} is responsible for generating {@link FetchRequest} that represent the + * {@link SubscriptionState#fetchablePartitions(Predicate)} based on the user's topic subscription/partition + * assignment. + */ +public class FetchRequestManager extends AbstractFetch implements RequestManager { + + private final NetworkClientDelegate networkClientDelegate; + private final List<CompletableFuture<Queue<CompletedFetch>>> futures; + + FetchRequestManager(final LogContext logContext, + final Time time, + final ConsumerMetadata metadata, + final SubscriptionState subscriptions, + final FetchConfig fetchConfig, + final FetchMetricsManager metricsManager, + final NetworkClientDelegate networkClientDelegate) { + super(logContext, metadata, subscriptions, fetchConfig, metricsManager, time); + this.networkClientDelegate = networkClientDelegate; + this.futures = new ArrayList<>(); + } + + @Override + protected boolean isUnavailable(Node node) { + return networkClientDelegate.isUnavailable(node); + } + + @Override + protected void maybeThrowAuthFailure(Node node) { + networkClientDelegate.maybeThrowAuthFailure(node); + } + + /** + * Adds a new {@link Future future} to the list of futures awaiting results. Per the comments on + * {@link #handleFetchResponse(Node, FetchSessionHandler.FetchRequestData, ClientResponse)}}, there is no + * guarantee that this particular future will be provided with a non-empty result, but it is guaranteed + * to be completed with a result, assuming that it does not time out. + * + * @param future Future that will be {@link CompletableFuture#complete(Object) completed} if not timed out + */ + public void requestFetch(CompletableFuture<Queue<CompletedFetch>> future) { + futures.add(future); + } + + /** + * {@inheritDoc} + */ + @Override + public PollResult poll(long currentTimeMs) { + return pollInternal( + prepareFetchRequests(), + this::handleFetchResponse, + this::handleFetchResponse + ); + } + + /** + * {@inheritDoc} + */ + @Override + public PollResult pollOnClose() { + return pollInternal( + prepareCloseFetchSessionRequests(), + this::handleCloseFetchSessionResponse, + this::handleCloseFetchSessionResponse + ); + } + + /** + * Creates the {@link PollResult poll result} that contains a list of zero or more + * {@link FetchRequest.Builder fetch requests} fetch request}, + * {@link NetworkClient#send(ClientRequest, long) enqueues/sends it, and adds the {@link RequestFuture callback} Review Comment: I removed the latter phrases to reduce confusion. ########## clients/src/main/java/org/apache/kafka/clients/consumer/internals/FetchRequestManager.java: ########## @@ -0,0 +1,181 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ +package org.apache.kafka.clients.consumer.internals; + +import java.util.ArrayList; +import java.util.LinkedList; +import java.util.List; +import java.util.Map; +import java.util.Queue; +import java.util.concurrent.CompletableFuture; +import java.util.concurrent.Future; +import java.util.function.BiConsumer; +import java.util.function.Predicate; +import java.util.stream.Collectors; + +import org.apache.kafka.clients.ClientRequest; +import org.apache.kafka.clients.ClientResponse; +import org.apache.kafka.clients.FetchSessionHandler; +import org.apache.kafka.clients.NetworkClient; +import org.apache.kafka.clients.consumer.internals.NetworkClientDelegate.PollResult; +import org.apache.kafka.clients.consumer.internals.NetworkClientDelegate.UnsentRequest; +import org.apache.kafka.common.Node; +import org.apache.kafka.common.requests.FetchRequest; +import org.apache.kafka.common.utils.LogContext; +import org.apache.kafka.common.utils.Time; + +/** + * {@code FetchRequestManager} is responsible for generating {@link FetchRequest} that represent the + * {@link SubscriptionState#fetchablePartitions(Predicate)} based on the user's topic subscription/partition + * assignment. + */ +public class FetchRequestManager extends AbstractFetch implements RequestManager { + + private final NetworkClientDelegate networkClientDelegate; + private final List<CompletableFuture<Queue<CompletedFetch>>> futures; + + FetchRequestManager(final LogContext logContext, + final Time time, + final ConsumerMetadata metadata, + final SubscriptionState subscriptions, + final FetchConfig fetchConfig, + final FetchMetricsManager metricsManager, + final NetworkClientDelegate networkClientDelegate) { + super(logContext, metadata, subscriptions, fetchConfig, metricsManager, time); + this.networkClientDelegate = networkClientDelegate; + this.futures = new ArrayList<>(); + } + + @Override + protected boolean isUnavailable(Node node) { + return networkClientDelegate.isUnavailable(node); + } + + @Override + protected void maybeThrowAuthFailure(Node node) { + networkClientDelegate.maybeThrowAuthFailure(node); + } + + /** + * Adds a new {@link Future future} to the list of futures awaiting results. Per the comments on + * {@link #handleFetchResponse(Node, FetchSessionHandler.FetchRequestData, ClientResponse)}}, there is no + * guarantee that this particular future will be provided with a non-empty result, but it is guaranteed + * to be completed with a result, assuming that it does not time out. + * + * @param future Future that will be {@link CompletableFuture#complete(Object) completed} if not timed out + */ + public void requestFetch(CompletableFuture<Queue<CompletedFetch>> future) { + futures.add(future); + } + + /** + * {@inheritDoc} + */ + @Override + public PollResult poll(long currentTimeMs) { + return pollInternal( + prepareFetchRequests(), + this::handleFetchResponse, + this::handleFetchResponse Review Comment: Agreed. I've updated the names. -- 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