You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@tez.apache.org by GitBox <gi...@apache.org> on 2021/08/06 00:04:59 UTC

[GitHub] [tez] rbalamohan commented on a change in pull request #123: TEZ-4231: SimpleHistoryParser doesn't merge events correctly

rbalamohan commented on a change in pull request #123:
URL: https://github.com/apache/tez/pull/123#discussion_r683841310



##########
File path: tez-tools/analyzers/job-analyzer/src/main/java/org/apache/tez/analyzer/plugins/TaskHangAnalyzer.java
##########
@@ -0,0 +1,137 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.tez.analyzer.plugins;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ToolRunner;
+import org.apache.tez.analyzer.Analyzer;
+import org.apache.tez.analyzer.CSVResult;
+import org.apache.tez.analyzer.Result;
+import org.apache.tez.dag.api.TezException;
+import org.apache.tez.history.parser.datamodel.DagInfo;
+import org.apache.tez.history.parser.datamodel.TaskAttemptInfo;
+import org.apache.tez.history.parser.datamodel.VertexInfo;
+
+import java.util.Comparator;
+import java.util.HashMap;
+import java.util.Map;
+
+/**
+ * Get the Task assignments on different nodes of the cluster.
+ */
+public class TaskHangAnalyzer extends TezAnalyzerBase implements Analyzer {
+  private final String[] headers = { "vertex", "task", " number_of_attempts", "last_attempt_id",
+      "last_attempt_status", "last_attempt_duration_ms", "last_attempt_node" };
+  private final CSVResult csvResult;
+
+  public TaskHangAnalyzer(Configuration config) {
+    super(config);
+    csvResult = new CSVResult(headers);
+  }
+
+  @Override
+  public void analyze(DagInfo dagInfo) throws TezException {
+    Map<String, Map<String, String>> taskData = new HashMap<>(); // task attempt count per task
+    for (VertexInfo vertex : dagInfo.getVertices()) {
+      taskData.clear();
+      for (TaskAttemptInfo attempt : vertex.getTaskAttempts()) {
+        String taskId = attempt.getTaskInfo().getTaskId();
+
+        int numAttemptsForTask = attempt.getTaskInfo().getNumberOfTaskAttempts();
+        Map<String, String> thisTaskData = taskData.get(taskId);
+
+        if (thisTaskData == null) {
+          thisTaskData = new HashMap<>();
+          thisTaskData.put("num_attempts", Integer.toString(numAttemptsForTask));
+          taskData.put(taskId, thisTaskData);
+        }
+
+        // attempt_1599682376162_0006_27_00_000086_1
+        int attemptNumber = Integer.parseInt(attempt.getTaskAttemptId().split("_")[6]);

Review comment:
       Replace with "TezTaskAttemptID.fromString(attempt.getTaskAttemptId()).getId()}" ?

##########
File path: tez-tools/analyzers/job-analyzer/src/main/java/org/apache/tez/analyzer/plugins/TaskHangAnalyzer.java
##########
@@ -0,0 +1,137 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.tez.analyzer.plugins;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ToolRunner;
+import org.apache.tez.analyzer.Analyzer;
+import org.apache.tez.analyzer.CSVResult;
+import org.apache.tez.analyzer.Result;
+import org.apache.tez.dag.api.TezException;
+import org.apache.tez.history.parser.datamodel.DagInfo;
+import org.apache.tez.history.parser.datamodel.TaskAttemptInfo;
+import org.apache.tez.history.parser.datamodel.VertexInfo;
+
+import java.util.Comparator;
+import java.util.HashMap;
+import java.util.Map;
+
+/**
+ * Get the Task assignments on different nodes of the cluster.
+ */
+public class TaskHangAnalyzer extends TezAnalyzerBase implements Analyzer {
+  private final String[] headers = { "vertex", "task", " number_of_attempts", "last_attempt_id",
+      "last_attempt_status", "last_attempt_duration_ms", "last_attempt_node" };
+  private final CSVResult csvResult;
+
+  public TaskHangAnalyzer(Configuration config) {
+    super(config);
+    csvResult = new CSVResult(headers);
+  }
+
+  @Override
+  public void analyze(DagInfo dagInfo) throws TezException {
+    Map<String, Map<String, String>> taskData = new HashMap<>(); // task attempt count per task
+    for (VertexInfo vertex : dagInfo.getVertices()) {
+      taskData.clear();
+      for (TaskAttemptInfo attempt : vertex.getTaskAttempts()) {
+        String taskId = attempt.getTaskInfo().getTaskId();
+
+        int numAttemptsForTask = attempt.getTaskInfo().getNumberOfTaskAttempts();
+        Map<String, String> thisTaskData = taskData.get(taskId);
+
+        if (thisTaskData == null) {
+          thisTaskData = new HashMap<>();
+          thisTaskData.put("num_attempts", Integer.toString(numAttemptsForTask));
+          taskData.put(taskId, thisTaskData);
+        }
+
+        // attempt_1599682376162_0006_27_00_000086_1
+        int attemptNumber = Integer.parseInt(attempt.getTaskAttemptId().split("_")[6]);
+        if (attemptNumber == numAttemptsForTask - 1) {
+          thisTaskData.put("last_attempt_id", attempt.getTaskAttemptId());
+          thisTaskData.put("last_attempt_status", attempt.getDetailedStatus());
+          thisTaskData.put("last_attempt_node", attempt.getNodeId());
+
+          thisTaskData.put("last_attempt_duration_ms",
+              (attempt.getFinishTime() == 0 || attempt.getStartTime() == 0) ? "-1"
+                : Long.toString(attempt.getFinishTime() - attempt.getStartTime()));
+        }
+      }
+      for (Map.Entry<String, Map<String, String>> task : taskData.entrySet()) {
+        addARecord(vertex.getVertexName(), task.getKey(), task.getValue().get("num_attempts"),
+            task.getValue().get("last_attempt_id"), task.getValue().get("last_attempt_status"),
+            task.getValue().get("last_attempt_duration_ms"),
+            task.getValue().get("last_attempt_node"));
+      }
+    }
+
+    csvResult.sort(new Comparator<String[]>() {
+      public int compare(String[] first, String[] second) {
+        int vertexOrder = first[0].compareTo(second[0]);
+        int lastAttemptStatusOrder =
+            (first[4] == null || second[4] == null) ? 0 : first[4].compareTo(second[4]);
+        int attemptNumberOrder = Integer.valueOf(second[2]).compareTo(Integer.valueOf(first[2]));
+
+        return vertexOrder == 0
+          ? (lastAttemptStatusOrder == 0 ? attemptNumberOrder : lastAttemptStatusOrder)
+          : vertexOrder;
+      }
+    });
+  }
+
+  private void addARecord(String vertexName, String taskId, String numAttempts,
+      String lastAttemptId, String lastAttemptStatus, String lastAttemptDuration,
+      String lastAttemptNode) {
+    String[] record = new String[7];
+    record[0] = vertexName;
+    record[1] = taskId;
+    record[2] = numAttempts;
+    record[3] = lastAttemptId;
+    record[4] = lastAttemptStatus;
+    record[5] = lastAttemptDuration;
+    record[6] = lastAttemptNode;
+
+    csvResult.addRecord(record);
+  }
+
+  @Override
+  public Result getResult() throws TezException {
+    return csvResult;
+  }
+
+  @Override
+  public String getName() {
+    return "Task Hang Analyzer";
+  }
+
+  @Override
+  public String getDescription() {
+    return "TaskHandAnalyzer can give quick insights about hanging tasks/task attempts"

Review comment:
       Typo?

##########
File path: tez-tools/analyzers/job-analyzer/src/main/java/org/apache/tez/analyzer/plugins/TaskHangAnalyzer.java
##########
@@ -0,0 +1,137 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.tez.analyzer.plugins;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ToolRunner;
+import org.apache.tez.analyzer.Analyzer;
+import org.apache.tez.analyzer.CSVResult;
+import org.apache.tez.analyzer.Result;
+import org.apache.tez.dag.api.TezException;
+import org.apache.tez.history.parser.datamodel.DagInfo;
+import org.apache.tez.history.parser.datamodel.TaskAttemptInfo;
+import org.apache.tez.history.parser.datamodel.VertexInfo;
+
+import java.util.Comparator;
+import java.util.HashMap;
+import java.util.Map;
+
+/**
+ * Get the Task assignments on different nodes of the cluster.

Review comment:
       Fix comments as this analyser is related to hung task analysis

##########
File path: tez-tools/analyzers/job-analyzer/src/main/java/org/apache/tez/analyzer/plugins/TaskHangAnalyzer.java
##########
@@ -0,0 +1,137 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.tez.analyzer.plugins;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ToolRunner;
+import org.apache.tez.analyzer.Analyzer;
+import org.apache.tez.analyzer.CSVResult;
+import org.apache.tez.analyzer.Result;
+import org.apache.tez.dag.api.TezException;
+import org.apache.tez.history.parser.datamodel.DagInfo;
+import org.apache.tez.history.parser.datamodel.TaskAttemptInfo;
+import org.apache.tez.history.parser.datamodel.VertexInfo;
+
+import java.util.Comparator;
+import java.util.HashMap;
+import java.util.Map;
+
+/**
+ * Get the Task assignments on different nodes of the cluster.
+ */
+public class TaskHangAnalyzer extends TezAnalyzerBase implements Analyzer {
+  private final String[] headers = { "vertex", "task", " number_of_attempts", "last_attempt_id",
+      "last_attempt_status", "last_attempt_duration_ms", "last_attempt_node" };
+  private final CSVResult csvResult;
+
+  public TaskHangAnalyzer(Configuration config) {
+    super(config);
+    csvResult = new CSVResult(headers);
+  }
+
+  @Override
+  public void analyze(DagInfo dagInfo) throws TezException {
+    Map<String, Map<String, String>> taskData = new HashMap<>(); // task attempt count per task
+    for (VertexInfo vertex : dagInfo.getVertices()) {
+      taskData.clear();
+      for (TaskAttemptInfo attempt : vertex.getTaskAttempts()) {
+        String taskId = attempt.getTaskInfo().getTaskId();
+
+        int numAttemptsForTask = attempt.getTaskInfo().getNumberOfTaskAttempts();
+        Map<String, String> thisTaskData = taskData.get(taskId);
+
+        if (thisTaskData == null) {
+          thisTaskData = new HashMap<>();
+          thisTaskData.put("num_attempts", Integer.toString(numAttemptsForTask));
+          taskData.put(taskId, thisTaskData);
+        }
+
+        // attempt_1599682376162_0006_27_00_000086_1
+        int attemptNumber = Integer.parseInt(attempt.getTaskAttemptId().split("_")[6]);
+        if (attemptNumber == numAttemptsForTask - 1) {

Review comment:
       Sometimes all the attempts may get scheduled on the same node and fail. It will be good to understand that as well. Would you like to refactor it such that, it can provide all id/status and respective node details?. 
   
   While providing final detail, it can be a concatenated string as well (to make it readable and printable easily)
   
   It will be nice to provide the info in this analyzer itself; (Without this info, user may have to co-relate with results of TaskAssignmentAnalyzer for node analysis). 

##########
File path: tez-tools/analyzers/job-analyzer/src/main/java/org/apache/tez/analyzer/plugins/TaskHangAnalyzer.java
##########
@@ -0,0 +1,137 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.tez.analyzer.plugins;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ToolRunner;
+import org.apache.tez.analyzer.Analyzer;
+import org.apache.tez.analyzer.CSVResult;
+import org.apache.tez.analyzer.Result;
+import org.apache.tez.dag.api.TezException;
+import org.apache.tez.history.parser.datamodel.DagInfo;
+import org.apache.tez.history.parser.datamodel.TaskAttemptInfo;
+import org.apache.tez.history.parser.datamodel.VertexInfo;
+
+import java.util.Comparator;
+import java.util.HashMap;
+import java.util.Map;
+
+/**
+ * Get the Task assignments on different nodes of the cluster.
+ */
+public class TaskHangAnalyzer extends TezAnalyzerBase implements Analyzer {

Review comment:
       Rename as "HungTaskAnalyzer" ?

##########
File path: tez-tools/analyzers/job-analyzer/src/main/java/org/apache/tez/analyzer/plugins/DagOverviewAnalyzer.java
##########
@@ -0,0 +1,127 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.tez.analyzer.plugins;
+
+import java.text.SimpleDateFormat;
+import java.util.Comparator;
+import java.util.Date;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ToolRunner;
+import org.apache.tez.analyzer.Analyzer;
+import org.apache.tez.analyzer.CSVResult;
+import org.apache.tez.analyzer.Result;
+import org.apache.tez.dag.api.TezException;
+import org.apache.tez.history.parser.datamodel.DagInfo;
+import org.apache.tez.history.parser.datamodel.Event;
+import org.apache.tez.history.parser.datamodel.TaskAttemptInfo;
+import org.apache.tez.history.parser.datamodel.TaskInfo;
+import org.apache.tez.history.parser.datamodel.VertexInfo;
+
+public class DagOverviewAnalyzer extends TezAnalyzerBase implements Analyzer {
+  private final String[] headers =
+      { "name", "id", "event_type", "status", "event_time", "event_time_str", "diagnostics" };

Review comment:
       Would it be possible to include the number of tasks assigned in the vertex as well (can be added in another field called "comments" or "additional info" which can be populated optionally).
   e.g "numTasks: " vertex.getNumTasks() + ", failedTasks: " + vertex.getFailedTasks().size()
             + ", completedTasks: " + vertex.getCompletedTasksCount()

##########
File path: tez-tools/analyzers/job-analyzer/src/main/java/org/apache/tez/analyzer/plugins/TaskHangAnalyzer.java
##########
@@ -0,0 +1,137 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.tez.analyzer.plugins;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ToolRunner;
+import org.apache.tez.analyzer.Analyzer;
+import org.apache.tez.analyzer.CSVResult;
+import org.apache.tez.analyzer.Result;
+import org.apache.tez.dag.api.TezException;
+import org.apache.tez.history.parser.datamodel.DagInfo;
+import org.apache.tez.history.parser.datamodel.TaskAttemptInfo;
+import org.apache.tez.history.parser.datamodel.VertexInfo;
+
+import java.util.Comparator;
+import java.util.HashMap;
+import java.util.Map;
+
+/**
+ * Get the Task assignments on different nodes of the cluster.
+ */
+public class TaskHangAnalyzer extends TezAnalyzerBase implements Analyzer {
+  private final String[] headers = { "vertex", "task", " number_of_attempts", "last_attempt_id",
+      "last_attempt_status", "last_attempt_duration_ms", "last_attempt_node" };
+  private final CSVResult csvResult;
+
+  public TaskHangAnalyzer(Configuration config) {
+    super(config);
+    csvResult = new CSVResult(headers);
+  }
+
+  @Override
+  public void analyze(DagInfo dagInfo) throws TezException {
+    Map<String, Map<String, String>> taskData = new HashMap<>(); // task attempt count per task
+    for (VertexInfo vertex : dagInfo.getVertices()) {
+      taskData.clear();
+      for (TaskAttemptInfo attempt : vertex.getTaskAttempts()) {
+        String taskId = attempt.getTaskInfo().getTaskId();
+
+        int numAttemptsForTask = attempt.getTaskInfo().getNumberOfTaskAttempts();
+        Map<String, String> thisTaskData = taskData.get(taskId);
+
+        if (thisTaskData == null) {
+          thisTaskData = new HashMap<>();
+          thisTaskData.put("num_attempts", Integer.toString(numAttemptsForTask));
+          taskData.put(taskId, thisTaskData);
+        }
+
+        // attempt_1599682376162_0006_27_00_000086_1
+        int attemptNumber = Integer.parseInt(attempt.getTaskAttemptId().split("_")[6]);
+        if (attemptNumber == numAttemptsForTask - 1) {
+          thisTaskData.put("last_attempt_id", attempt.getTaskAttemptId());

Review comment:
       Declare as static final Strings?

##########
File path: tez-plugins/tez-protobuf-history-plugin/src/main/java/org/apache/tez/dag/history/logging/proto/HistoryEventProtoJsonConversion.java
##########
@@ -556,12 +556,12 @@ private static JSONObject convertTaskFinishedEvent(HistoryEventProto event) thro
     events.put(finishEvent);
     jsonObject.put(ATSConstants.EVENTS, events);
 
-    long startTime = getLongDataValueByKey(event, ATSConstants.START_TIME);
+    long timeTaken = getLongDataValueByKey(event, ATSConstants.TIME_TAKEN);
 
     JSONObject otherInfo = new JSONObject();
-    otherInfo.put(ATSConstants.START_TIME, startTime);
+    otherInfo.put(ATSConstants.START_TIME, event.getEventTime() - timeTaken);

Review comment:
       IIRC, there were corner cases where events will not be properly populated. May be in cases, where vertices were shutdown due to errors or so (need to check). 
   
   In such cases, this would have returned "-ve" value earlier. 
   
   Current patch seem to change the start_time, depending on getEventTime.  This could give a perspective that the task/vertex was there for very short time. 
   
   Can you plz share more info on prev error? Were you getting -ve values earlier for which this is being modified? 

##########
File path: tez-plugins/tez-history-parser/src/main/java/org/apache/tez/history/parser/SimpleHistoryParser.java
##########
@@ -236,11 +236,14 @@ protected void readEventsFromSource(String dagId, JSONObjectSource source,
         // time etc).
         if (dagJson == null) {
           dagJson = jsonObject;
-        } else if (dagJson.optJSONObject(ATSConstants.OTHER_INFO)
-            .optJSONObject(ATSConstants.DAG_PLAN) == null) {
-          // if DAG_PLAN is not filled already, let's try to fetch it from other
-          dagJson.getJSONObject(ATSConstants.OTHER_INFO).put(ATSConstants.DAG_PLAN, jsonObject
-              .getJSONObject(ATSConstants.OTHER_INFO).getJSONObject(ATSConstants.DAG_PLAN));
+        } else{

Review comment:
       minor: fix indent

##########
File path: tez-tools/analyzers/job-analyzer/src/main/java/org/apache/tez/analyzer/plugins/InputReadErrorAnalyzer.java
##########
@@ -0,0 +1,94 @@
+/**
+ * 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
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * 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.tez.analyzer.plugins;
+
+import java.util.Comparator;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.util.ToolRunner;
+import org.apache.tez.analyzer.Analyzer;
+import org.apache.tez.analyzer.CSVResult;
+import org.apache.tez.analyzer.Result;
+import org.apache.tez.dag.api.TezException;
+import org.apache.tez.history.parser.datamodel.DagInfo;
+import org.apache.tez.history.parser.datamodel.Event;
+import org.apache.tez.history.parser.datamodel.TaskAttemptInfo;
+import org.apache.tez.history.parser.datamodel.VertexInfo;
+
+/**
+ * This analyzer is support to collect which nodes can be blamed for shuffle read errors.
+ */
+public class InputReadErrorAnalyzer extends TezAnalyzerBase implements Analyzer {

Review comment:
       Nice!




-- 
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: issues-unsubscribe@tez.apache.org

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