You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by GitBox <gi...@apache.org> on 2022/03/22 00:25:36 UTC

[GitHub] [pinot] Jackie-Jiang commented on a change in pull request #8379: create routeTable and routeManager API

Jackie-Jiang commented on a change in pull request #8379:
URL: https://github.com/apache/pinot/pull/8379#discussion_r831643170



##########
File path: pinot-broker/src/main/java/org/apache/pinot/broker/routing/RoutingManager.java
##########
@@ -82,7 +83,7 @@
  * TODO: Expose RoutingEntry class to get a consistent view in the broker request handler and save the redundant map
  *       lookups.
  */
-public class RoutingManager implements ClusterChangeHandler {
+public class RoutingManager implements RouteManager, ClusterChangeHandler {

Review comment:
       Let's name the interface `RoutingManager` and rename the implementation to `DefaultRoutingManager`

##########
File path: pinot-broker/src/main/java/org/apache/pinot/broker/routing/RoutingTable.java
##########
@@ -20,10 +20,11 @@
 
 import java.util.List;
 import java.util.Map;
+import org.apache.pinot.core.routing.RouteTable;
 import org.apache.pinot.core.transport.ServerInstance;
 
 
-public class RoutingTable {
+public class RoutingTable implements RouteTable {

Review comment:
       Name the interface `RoutingTable` and rename the implementation to `RoutingTableImpl`

##########
File path: pinot-broker/src/main/java/org/apache/pinot/broker/routing/instanceselector/InstanceSelector.java
##########
@@ -56,7 +55,7 @@
    * to selected server instance hosting the segment and a set of unavailable segments (no enabled instance or all
    * enabled instances are in ERROR state).
    */
-  SelectionResult select(BrokerRequest brokerRequest, List<String> segments);
+  SelectionResult select(List<String> segments);

Review comment:
       Don't change this API. The future implementation of this might need the query to correctly select the instance

##########
File path: pinot-core/src/main/java/org/apache/pinot/core/routing/RouteManager.java
##########
@@ -0,0 +1,65 @@
+/**
+ * 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.pinot.core.routing;
+
+import java.util.Map;
+import org.apache.pinot.common.request.BrokerRequest;
+import org.apache.pinot.core.transport.ServerInstance;
+
+/**
+ * The {@code RouteManager} provides the routing information for a query that requests access to a Pinot table.
+ *
+ * The implementation of this interface should ensure the routing and server information are up-to-date at the
+ * time when the routing request was made.
+ *
+ * set by the user. This needs to be added to support features like segment pruning.
+ */
+public interface RouteManager {
+
+  /**
+   * Validate routing exist for a table
+   *
+   * @param tableNameWithType the name of the table.
+   * @return true if the route table exists.
+   */
+  boolean routingExists(String tableNameWithType);
+
+  /**
+   * Get all enabled server instances that are available for routing.
+   *
+   * @return all currently enabled server instances.
+   */
+  Map<String, ServerInstance> getEnabledServerInstanceMap();
+
+  /**
+   * Get the {@link RouteTable} for a specified table.
+   *
+   * @param tableNameWithType the name of the table.
+   * @return the route table.
+   */
+  RouteTable getRoutingTable(String tableNameWithType);

Review comment:
       We can provide a default implementation to compile `SELECT * FROM <tableNameWithType>` and call `getRoutingTable()` with `brokerRequest`

##########
File path: pinot-broker/src/main/java/org/apache/pinot/broker/routing/RoutingManager.java
##########
@@ -458,13 +460,42 @@ public boolean routingExists(String tableNameWithType) {
    * <p>NOTE: The broker request should already have the table suffix (_OFFLINE or _REALTIME) appended.
    */
   @Nullable
+  @Override
   public RoutingTable getRoutingTable(BrokerRequest brokerRequest) {
     String tableNameWithType = brokerRequest.getQuerySource().getTableName();
     RoutingEntry routingEntry = _routingEntryMap.get(tableNameWithType);
     if (routingEntry == null) {
       return null;
     }
     InstanceSelector.SelectionResult selectionResult = routingEntry.calculateRouting(brokerRequest);
+    return constructRoutingTableFromInstanceSelectionResult(tableNameWithType, selectionResult);
+  }
+
+  @Override
+  public RoutingTable getRoutingTable(String tableNameWithType) {
+    RoutingEntry routingEntry = _routingEntryMap.get(tableNameWithType);
+    if (routingEntry == null) {
+      return null;
+    }
+    InstanceSelector.SelectionResult selectionResult = routingEntry.calculateRouting(tableNameWithType);
+    return constructRoutingTableFromInstanceSelectionResult(tableNameWithType, selectionResult);
+  }
+
+  @Override
+  public Map<String, ServerInstance> getEnabledServerInstanceMap() {
+    return new HashMap<>(_enabledServerInstanceMap);

Review comment:
       Don't make a copy as it can be expensive, just return as is since we can control the caller code




-- 
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: commits-unsubscribe@pinot.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org