You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by ma...@apache.org on 2021/05/11 03:42:08 UTC

[incubator-pinot] branch master updated: Add debug endpoint for tables. (#6897)

This is an automated email from the ASF dual-hosted git repository.

mayanks pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-pinot.git


The following commit(s) were added to refs/heads/master by this push:
     new ff8700f  Add debug endpoint for tables. (#6897)
ff8700f is described below

commit ff8700ff33b45dcecdbf3912bae11853487cde9e
Author: Mayank Shrivastava <ma...@apache.org>
AuthorDate: Mon May 10 20:41:46 2021 -0700

    Add debug endpoint for tables. (#6897)
    
    When debugging issues in production, there's a list of commonly used debugging steps
    to indentify the issue. In this PR, we add an endpoint which automates this process
    and provides a summary of possible issues.
    
    - Endpoint added on controller: `debug/tables/{tableName}?type={offline|realtime}
    - Enhancements to add:
      - Add ingestions status for realtime tables (reusing #6890, when merged).
      - Error surfacing
    - Sample JSON output:
    
    ```
    [ {
      "tableName" : "airlineStats_OFFLINE",
      "numSegments" : 31,
      "numServers" : 1,
      "numBrokers" : 1,
      "segmentDebugInfos" : [ {
        "segmentName" : "airlineStats_OFFLINE_16071_16071_0",
        "segmentStateInServer" : [ {
          "serverName" : "Server_192.168.1.90_7000",
          "idealStateStatus" : "ONLINE",
          "externalViewStatus" : "ERROR"
        } ]
      }, {
        "segmentName" : "airlineStats_OFFLINE_16095_16095_0",
        "segmentStateInServer" : [ {
          "serverName" : "Server_192.168.1.90_7000",
          "idealStateStatus" : "ONLINE",
          "externalViewStatus" : "ERROR"
        } ]
      }, {
        "segmentName" : "airlineStats_OFFLINE_16100_16100_0",
        "segmentStateInServer" : [ {
          "serverName" : "Server_192.168.1.90_7000",
          "idealStateStatus" : "ONLINE",
          "externalViewStatus" : "ERROR"
        } ]
      } ],
      "serverDebugInfos" : [ {
        "errors" : 31,
        "serverName" : "Server_192.168.1.90_7000",
        "numReadMessages" : 0,
        "numNewMessages" : 0
      } ],
      "brokerDebugInfos" : [ ],
      "tableSize" : {
        "reportedSize" : "3 MB",
        "estimatedSize" : "3 MB"
      }
    }, {
      "tableName" : "airlineStats_REALTIME",
      "numSegments" : 1,
      "numServers" : 1,
      "numBrokers" : 1,
      "segmentDebugInfos" : [ ],
      "serverDebugInfos" : [ {
        "errors" : 0,
        "serverName" : "Server_192.168.1.90_7000",
        "numReadMessages" : 0,
        "numNewMessages" : 0
      } ],
      "brokerDebugInfos" : [ ],
      "tableSize" : {
        "reportedSize" : "1 MB",
        "estimatedSize" : "1 MB"
      }
    } ]
    ```
---
 .../pinot/controller/api/debug/TableDebugInfo.java | 207 +++++++++++++++++
 .../api/resources/TableDebugResource.java          | 256 +++++++++++++++++++++
 2 files changed, 463 insertions(+)

diff --git a/pinot-controller/src/main/java/org/apache/pinot/controller/api/debug/TableDebugInfo.java b/pinot-controller/src/main/java/org/apache/pinot/controller/api/debug/TableDebugInfo.java
new file mode 100644
index 0000000..a9bc6bc
--- /dev/null
+++ b/pinot-controller/src/main/java/org/apache/pinot/controller/api/debug/TableDebugInfo.java
@@ -0,0 +1,207 @@
+/**
+ * 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.controller.api.debug;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonIgnoreProperties;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.annotation.JsonPropertyOrder;
+import java.util.List;
+import org.apache.commons.io.FileUtils;
+
+
+@JsonPropertyOrder({"tableName", "numSegments", "numServers", "numBrokers", "segmentDebugInfos", "serverDebugInfos", "brokerDebugInfos"})
+@JsonIgnoreProperties(ignoreUnknown = true)
+@SuppressWarnings("unused")
+public class TableDebugInfo {
+  @JsonProperty("tableName")
+  private final String _tableName;
+
+  @JsonProperty("tableSize")
+  private final TableSizeSummary _tableSizeSummary;
+
+  @JsonProperty("numSegments")
+  private final int _numSegments;
+
+  @JsonProperty("numServers")
+  private final int _numServers;
+
+  @JsonProperty("numBrokers")
+  private final int _numBrokers;
+
+  @JsonProperty("segmentDebugInfos")
+  private final List<SegmentDebugInfo> _segmentDebugInfos;
+
+  @JsonProperty("serverDebugInfos")
+  private final List<ServerDebugInfo> _serverDebugInfos;
+
+  @JsonProperty("brokerDebugInfos")
+  private final List<BrokerDebugInfo> _brokerDebugInfos;
+
+  @JsonCreator
+  public TableDebugInfo(String tableName, TableSizeSummary tableSizeSummary, int numBrokers, int numServers,
+      int numSegments, List<SegmentDebugInfo> segmentDebugInfos, List<ServerDebugInfo> serverDebugInfos,
+      List<BrokerDebugInfo> brokerDebugInfos) {
+    _tableName = tableName;
+    _tableSizeSummary = tableSizeSummary;
+
+    _numBrokers = numBrokers;
+    _numServers = numServers;
+    _numSegments = numSegments;
+
+    _segmentDebugInfos = segmentDebugInfos;
+    _serverDebugInfos = serverDebugInfos;
+    _brokerDebugInfos = brokerDebugInfos;
+  }
+
+  public String getTableName() {
+    return _tableName;
+  }
+
+  public TableSizeSummary getTableSize() {
+    return _tableSizeSummary;
+  }
+
+  public int getNumSegments() {
+    return _numSegments;
+  }
+
+  public int getNumServers() {
+    return _numServers;
+  }
+
+  public int getNumBrokers() {
+    return _numBrokers;
+  }
+
+  public List<SegmentDebugInfo> getSegmentDebugInfos() {
+    return _segmentDebugInfos;
+  }
+
+  public List<ServerDebugInfo> getServerDebugInfos() {
+    return _serverDebugInfos;
+  }
+
+  public List<BrokerDebugInfo> getBrokerDebugInfos() {
+    return _brokerDebugInfos;
+  }
+
+  public static class SegmentDebugInfo {
+    private final String _segmentName;
+    private final List<IsEvState> _states;
+
+    public SegmentDebugInfo(String name, List<IsEvState> states) {
+      _segmentName = name;
+      _states = states;
+    }
+
+    public String getSegmentName() {
+      return _segmentName;
+    }
+
+    public List<IsEvState> getSegmentStateInServer() {
+      return _states;
+    }
+  }
+
+  public static class IsEvState {
+    private final String _serverName;
+    private final String _idealStateStatus;
+    private final String _externalViewStatus;
+
+    public IsEvState(String name, String idealStateStatus, String externalViewStatus) {
+      _serverName = name;
+      _idealStateStatus = idealStateStatus;
+      _externalViewStatus = externalViewStatus;
+    }
+
+    public String getServerName() {
+      return _serverName;
+    }
+
+    public String getIdealStateStatus() {
+      return _idealStateStatus;
+    }
+
+    public String getExternalViewStatus() {
+      return _externalViewStatus;
+    }
+  }
+
+  public static class ServerDebugInfo {
+    private final int _numErrors;
+    private final int _numMessages;
+    private final String _serverName;
+
+    public ServerDebugInfo(String serverName, int numErrors, int numMessages) {
+      _serverName = serverName;
+      _numErrors = numErrors;
+      _numMessages = numMessages;
+    }
+
+    public int getErrors() {
+      return _numErrors;
+    }
+
+    public int getNumMessages() {
+      return _numMessages;
+    }
+
+    public String getServerName() {
+      return _serverName;
+    }
+  }
+
+  public static class BrokerDebugInfo {
+    private final String _brokerName;
+    private final IsEvState _state;
+
+    public BrokerDebugInfo(String brokerName, IsEvState state) {
+      _brokerName = brokerName;
+      _state = state;
+    }
+
+    public String getBrokerName() {
+      return _brokerName;
+    }
+
+    public IsEvState getState() {
+      return _state;
+    }
+  }
+
+  public static class TableSizeSummary {
+    private final String _reportedSize;
+    private final String _estimatedSize;
+
+    public TableSizeSummary(long reportedSize, long estimatedSize) {
+
+      _reportedSize = FileUtils.byteCountToDisplaySize(reportedSize);
+      _estimatedSize = FileUtils.byteCountToDisplaySize(estimatedSize);
+    }
+
+    public String getReportedSize() {
+      return _reportedSize;
+    }
+
+    public String getEstimatedSize() {
+      return _estimatedSize;
+    }
+  }
+}
diff --git a/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/TableDebugResource.java b/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/TableDebugResource.java
new file mode 100644
index 0000000..91ca171
--- /dev/null
+++ b/pinot-controller/src/main/java/org/apache/pinot/controller/api/resources/TableDebugResource.java
@@ -0,0 +1,256 @@
+/**
+ * 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.controller.api.resources;
+
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import com.fasterxml.jackson.databind.node.ObjectNode;
+import io.swagger.annotations.Api;
+import io.swagger.annotations.ApiOperation;
+import io.swagger.annotations.ApiParam;
+import io.swagger.annotations.ApiResponse;
+import io.swagger.annotations.ApiResponses;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.List;
+import java.util.Map;
+import java.util.concurrent.Executor;
+import javax.inject.Inject;
+import javax.ws.rs.GET;
+import javax.ws.rs.Path;
+import javax.ws.rs.PathParam;
+import javax.ws.rs.Produces;
+import javax.ws.rs.QueryParam;
+import javax.ws.rs.core.MediaType;
+import org.apache.commons.httpclient.HttpConnectionManager;
+import org.apache.helix.HelixDataAccessor;
+import org.apache.helix.HelixProperty;
+import org.apache.helix.PropertyKey;
+import org.apache.helix.model.ExternalView;
+import org.apache.helix.model.IdealState;
+import org.apache.pinot.common.metrics.ControllerMetrics;
+import org.apache.pinot.controller.ControllerConf;
+import org.apache.pinot.controller.api.debug.TableDebugInfo;
+import org.apache.pinot.controller.helix.core.PinotHelixResourceManager;
+import org.apache.pinot.controller.util.TableSizeReader;
+import org.apache.pinot.spi.config.table.TableType;
+import org.apache.pinot.spi.utils.JsonUtils;
+import org.apache.pinot.spi.utils.builder.TableNameBuilder;
+
+import static org.apache.pinot.spi.utils.CommonConstants.Helix.BROKER_RESOURCE_INSTANCE;
+
+
+/**
+ * This class implements the debugging endpoints.
+ */
+@Api(tags = Constants.CLUSTER_TAG)
+@Path("/")
+public class TableDebugResource {
+  @Inject
+  PinotHelixResourceManager _pinotHelixResourceManager;
+
+  @Inject
+  Executor _executor;
+  @Inject
+  HttpConnectionManager _connectionManager;
+
+  @Inject
+  ControllerMetrics _controllerMetrics;
+
+  @Inject
+  ControllerConf _controllerConf;
+
+  @GET
+  @Path("/debug/tables/{tableName}")
+  @Produces(MediaType.APPLICATION_JSON)
+  @ApiOperation(value = "Get debug information for table.", notes = "Debug information for table.")
+  @ApiResponses(value = {@ApiResponse(code = 200, message = "Success"), @ApiResponse(code = 500, message = "Internal server error")})
+  public String getClusterInfo(
+      @ApiParam(value = "Name of the table", required = true) @PathParam("tableName") String tableName,
+      @ApiParam(value = "OFFLINE|REALTIME") @QueryParam("type") String tableTypeStr)
+      throws JsonProcessingException {
+    ObjectNode root = JsonUtils.newObjectNode();
+    root.put("clusterName", _pinotHelixResourceManager.getHelixClusterName());
+
+    TableType tableType = Constants.validateTableType(tableTypeStr);
+    List<TableType> tableTypes = (tableType == null) ? Arrays.asList(TableType.OFFLINE, TableType.REALTIME)
+        : Collections.singletonList(tableType);
+
+    List<TableDebugInfo> tableDebugInfos = new ArrayList<>();
+    for (TableType type : tableTypes) {
+      tableDebugInfos.add(debugTable(_pinotHelixResourceManager, tableName, type));
+    }
+    return new ObjectMapper().writerWithDefaultPrettyPrinter().writeValueAsString(tableDebugInfos);
+  }
+
+  /**
+   * Helper method to colelct debug information about the table.
+   *
+   * @param pinotHelixResourceManager Helix Resource Manager for the cluster
+   * @param tableName Name of the table
+   * @param tableType Type of the table (offline|realtime)
+   * @return TableDebugInfo
+   */
+  private TableDebugInfo debugTable(PinotHelixResourceManager pinotHelixResourceManager, String tableName,
+      TableType tableType) {
+    String tableNameWithType = TableNameBuilder.forType(tableType).tableNameWithType(tableName);
+
+    // Debug information for segments of the table.
+    List<TableDebugInfo.SegmentDebugInfo> segmentDebugInfos =
+        debugSegments(pinotHelixResourceManager, tableNameWithType);
+
+    // Debug information from brokers of the table.
+    List<TableDebugInfo.BrokerDebugInfo> brokerDebugInfos = debugBrokers(tableNameWithType);
+
+    // Debug information from servers of the table.
+    List<TableDebugInfo.ServerDebugInfo> serverDebugInfos =
+        debugServers(pinotHelixResourceManager, tableName, tableType);
+
+    // Table size summary.
+    TableDebugInfo.TableSizeSummary tableSizeSummary = getTableSize(tableNameWithType);
+
+    // Number of segments in the table.
+    IdealState idealState = _pinotHelixResourceManager.getTableIdealState(tableNameWithType);
+    int numSegments = (idealState != null) ? idealState.getPartitionSet().size() : 0;
+
+    return new TableDebugInfo(tableNameWithType, tableSizeSummary,
+        _pinotHelixResourceManager.getBrokerInstancesForTable(tableName, tableType).size(),
+        _pinotHelixResourceManager.getServerInstancesForTable(tableName, tableType).size(), numSegments,
+        segmentDebugInfos, serverDebugInfos, brokerDebugInfos);
+  }
+
+  private TableDebugInfo.TableSizeSummary getTableSize(String tableNameWithType) {
+    TableSizeReader tableSizeReader =
+        new TableSizeReader(_executor, _connectionManager, _controllerMetrics, _pinotHelixResourceManager);
+    TableSizeReader.TableSizeDetails tableSizeDetails;
+    try {
+      tableSizeDetails = tableSizeReader
+          .getTableSizeDetails(tableNameWithType, _controllerConf.getServerAdminRequestTimeoutSeconds() * 1000);
+    } catch (Throwable t) {
+      tableSizeDetails = null;
+    }
+
+    return (tableSizeDetails != null) ? new TableDebugInfo.TableSizeSummary(tableSizeDetails.reportedSizeInBytes,
+        tableSizeDetails.estimatedSizeInBytes) : new TableDebugInfo.TableSizeSummary(-1, -1);
+  }
+
+  /**
+   * Helper method to debug segments. Computes differences between ideal state and external view for each segment.
+   *
+   * @param pinotHelixResourceManager Helix Resource Manager
+   * @param tableNameWithType Name of table with type
+   * @return Debug information for segments
+   */
+  private List<TableDebugInfo.SegmentDebugInfo> debugSegments(PinotHelixResourceManager pinotHelixResourceManager,
+      String tableNameWithType) {
+    ExternalView externalView = pinotHelixResourceManager.getTableExternalView(tableNameWithType);
+    IdealState idealState = pinotHelixResourceManager.getTableIdealState(tableNameWithType);
+
+    List<TableDebugInfo.SegmentDebugInfo> segmentDebugInfos = new ArrayList<>();
+    if (idealState == null) {
+      return segmentDebugInfos;
+    }
+
+    for (Map.Entry<String, Map<String, String>> segmentMapEntry : idealState.getRecord().getMapFields().entrySet()) {
+      String segment = segmentMapEntry.getKey();
+      Map<String, String> instanceStateMap = segmentMapEntry.getValue();
+
+      List<TableDebugInfo.IsEvState> isEvStates = new ArrayList<>();
+      for (Map.Entry<String, String> entry : instanceStateMap.entrySet()) {
+        String serverName = entry.getKey();
+        String isState = entry.getValue();
+
+        String evState = (externalView != null) ? externalView.getStateMap(segment).get(serverName) : "null";
+        if (!isState.equals(evState)) {
+          isEvStates.add(new TableDebugInfo.IsEvState(serverName, isState, evState));
+        }
+      }
+
+      if (isEvStates.size() > 0) {
+        segmentDebugInfos.add(new TableDebugInfo.SegmentDebugInfo(segment, isEvStates));
+      }
+    }
+    return segmentDebugInfos;
+  }
+
+  /**
+   * Computes debug information for brokers. Identifies differences in ideal state and external view
+   * for the broker resource.
+   *
+   * @param tableNameWithType Name of table with type suffix
+   * @return Debug information for brokers of the table
+   */
+  private List<TableDebugInfo.BrokerDebugInfo> debugBrokers(String tableNameWithType) {
+    List<TableDebugInfo.BrokerDebugInfo> brokerDebugInfos = new ArrayList<>();
+    HelixDataAccessor helixDataAccessor = _pinotHelixResourceManager.getHelixZkManager().getHelixDataAccessor();
+    IdealState idealState =
+        helixDataAccessor.getProperty(helixDataAccessor.keyBuilder().idealStates(BROKER_RESOURCE_INSTANCE));
+    ExternalView externalView =
+        helixDataAccessor.getProperty(helixDataAccessor.keyBuilder().externalView(BROKER_RESOURCE_INSTANCE));
+
+    for (Map.Entry<String, String> entry : idealState.getInstanceStateMap(tableNameWithType).entrySet()) {
+      String brokerName = entry.getKey();
+      String isState = entry.getValue();
+      String evState = externalView.getStateMap(tableNameWithType).get(brokerName);
+      if (!isState.equals(evState)) {
+        brokerDebugInfos.add(
+            new TableDebugInfo.BrokerDebugInfo(brokerName, new TableDebugInfo.IsEvState(brokerName, isState, evState)));
+      }
+    }
+    return brokerDebugInfos;
+  }
+
+  /**
+   * Computes debug information for servers.
+   *
+   * @param pinotHelixResourceManager Helix Resource Manager
+   * @param tableName Name of table
+   * @param tableType Type of table (offline|realtime)
+   * @return Debug information for servers of the table
+   */
+  private List<TableDebugInfo.ServerDebugInfo> debugServers(PinotHelixResourceManager pinotHelixResourceManager,
+      String tableName, TableType tableType) {
+    HelixDataAccessor accessor = _pinotHelixResourceManager.getHelixZkManager().getHelixDataAccessor();
+    List<TableDebugInfo.ServerDebugInfo> serverDebugInfos = new ArrayList<>();
+
+    for (String instanceName : pinotHelixResourceManager.getServerInstancesForTable(tableName, tableType)) {
+      PropertyKey.Builder keyBuilder = accessor.keyBuilder();
+      List<String> sessionIds = accessor.getChildNames(keyBuilder.errors(instanceName));
+
+      if (sessionIds == null || sessionIds.size() == 0) {
+        return serverDebugInfos;
+      }
+
+      int numErrors = 0;
+      String tableNameWithType = TableNameBuilder.forType(tableType).tableNameWithType(tableName);
+      for (String sessionId : sessionIds) {
+        List<HelixProperty> childValues =
+            accessor.getChildValues(keyBuilder.errors(instanceName, sessionId, tableNameWithType), false);
+        numErrors += ((childValues != null) ? childValues.size() : 0);
+      }
+
+      List<String> messageNames = accessor.getChildNames(accessor.keyBuilder().messages(instanceName));
+      int numMessages = (messageNames != null) ? messageNames.size() : 0;
+
+      serverDebugInfos.add(new TableDebugInfo.ServerDebugInfo(instanceName, numErrors, numMessages));
+    }
+    return serverDebugInfos;
+  }
+}

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