yashmayya commented on code in PR #16691:
URL: https://github.com/apache/pinot/pull/16691#discussion_r2300064083


##########
pinot-query-runtime/src/main/java/org/apache/pinot/query/mailbox/channel/ChannelManager.java:
##########
@@ -35,32 +39,70 @@
  * query/job/stages.
  */
 public class ChannelManager {
+  /**
+   * Map from (hostname, port) to the ManagedChannel with all known channels
+   */
   private final ConcurrentHashMap<Pair<String, Integer>, ManagedChannel> 
_channelMap = new ConcurrentHashMap<>();
   private final TlsConfig _tlsConfig;
+  /**
+   * The idle timeout for the channel, which cannot be disabled in gRPC.
+   *
+   * In general we want to prevent the channel from going idle, so that we 
don't have to re-establish the connection
+   * (including TLS negotiation) before sending any message, which increases 
the latency of the first query sent after a
+   * period of inactivity.
+   *
+   * This is why by default we set the idle timeout to twice the pinger period 
if a pinger is configured, so that the
+   * pinger can keep the channel alive. In case the pinger is not configured, 
we set the idle timeout to 30 minutes,
+   * which is the default value in the gRPC Java implementation.

Review Comment:
   This comment needs to be updated.



##########
pinot-query-runtime/src/main/java/org/apache/pinot/query/mailbox/channel/ChannelManager.java:
##########
@@ -35,32 +39,70 @@
  * query/job/stages.
  */
 public class ChannelManager {
+  /**
+   * Map from (hostname, port) to the ManagedChannel with all known channels
+   */
   private final ConcurrentHashMap<Pair<String, Integer>, ManagedChannel> 
_channelMap = new ConcurrentHashMap<>();
   private final TlsConfig _tlsConfig;
+  /**
+   * The idle timeout for the channel, which cannot be disabled in gRPC.
+   *
+   * In general we want to prevent the channel from going idle, so that we 
don't have to re-establish the connection
+   * (including TLS negotiation) before sending any message, which increases 
the latency of the first query sent after a
+   * period of inactivity.
+   *
+   * This is why by default we set the idle timeout to twice the pinger period 
if a pinger is configured, so that the
+   * pinger can keep the channel alive. In case the pinger is not configured, 
we set the idle timeout to 30 minutes,
+   * which is the default value in the gRPC Java implementation.
+   */
+  private final Duration _idleTimeout;
   private final int _maxInboundMessageSize;
 
-  public ChannelManager(@Nullable TlsConfig tlsConfig, int 
maxInboundMessageSize) {
+  public ChannelManager(@Nullable TlsConfig tlsConfig, int 
maxInboundMessageSize, Duration idleTimeout) {
     _tlsConfig = tlsConfig;
     _maxInboundMessageSize = maxInboundMessageSize;
+    _idleTimeout = idleTimeout;
   }
 
   public ManagedChannel getChannel(String hostname, int port) {
     // TODO: Revisit parameters
     if (_tlsConfig != null) {
       return _channelMap.computeIfAbsent(Pair.of(hostname, port),
-          (k) -> NettyChannelBuilder
-              .forAddress(k.getLeft(), k.getRight())
-              .maxInboundMessageSize(_maxInboundMessageSize)
-              .sslContext(ServerGrpcQueryClient.buildSslContext(_tlsConfig))
-              .build()
+          (k) -> {
+            NettyChannelBuilder channelBuilder = NettyChannelBuilder
+                .forAddress(k.getLeft(), k.getRight())
+                .maxInboundMessageSize(
+                    _maxInboundMessageSize)
+                .sslContext(ServerGrpcQueryClient.buildSslContext(_tlsConfig));
+            return decorate(channelBuilder).build();
+          }
       );
     } else {
       return _channelMap.computeIfAbsent(Pair.of(hostname, port),
-          (k) -> ManagedChannelBuilder
-              .forAddress(k.getLeft(), k.getRight())
-              .maxInboundMessageSize(_maxInboundMessageSize)
-              .usePlaintext()
-              .build());
+          (k) -> {
+            ManagedChannelBuilder<?> channelBuilder = ManagedChannelBuilder
+                .forAddress(k.getLeft(), k.getRight())
+                .maxInboundMessageSize(
+                    _maxInboundMessageSize)
+                .usePlaintext();
+            return decorate(channelBuilder).build();
+          });
     }
   }
+
+  private ManagedChannelBuilder<?> decorate(ManagedChannelBuilder<?> builder) {
+    return builder.idleTimeout(_idleTimeout.getSeconds(), TimeUnit.SECONDS);
+  }
+
+  /**
+   * Returns a set with the key of all known channels.
+   *
+   * For each entry, the key is a pair of (hostname, port).
+   *
+   * The returned value is a copy of the internal map's key set, so it is safe 
to use
+   * without worrying about concurrent modifications.
+   */
+  public Set<Pair<String, Integer>> getKnownChannels() {

Review Comment:
   Unused now, can be removed?



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


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to