You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@tez.apache.org by hi...@apache.org on 2013/10/02 01:02:46 UTC

git commit: TEZ-493. Task JVMs hanging around even after the AM/app has completed. (hitesh)

Updated Branches:
  refs/heads/master ee5d5f87d -> 072dc8467


TEZ-493. Task JVMs hanging around even after the AM/app has completed. (hitesh)


Project: http://git-wip-us.apache.org/repos/asf/incubator-tez/repo
Commit: http://git-wip-us.apache.org/repos/asf/incubator-tez/commit/072dc846
Tree: http://git-wip-us.apache.org/repos/asf/incubator-tez/tree/072dc846
Diff: http://git-wip-us.apache.org/repos/asf/incubator-tez/diff/072dc846

Branch: refs/heads/master
Commit: 072dc84671ef89e5438c03de6cbaf220dc118e10
Parents: ee5d5f8
Author: Hitesh Shah <hi...@apache.org>
Authored: Tue Oct 1 16:02:32 2013 -0700
Committer: Hitesh Shah <hi...@apache.org>
Committed: Tue Oct 1 16:02:32 2013 -0700

----------------------------------------------------------------------
 .../apache/hadoop/mapred/YarnTezDagChild.java   | 55 ++++++++++----------
 1 file changed, 27 insertions(+), 28 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/incubator-tez/blob/072dc846/tez-dag/src/main/java/org/apache/hadoop/mapred/YarnTezDagChild.java
----------------------------------------------------------------------
diff --git a/tez-dag/src/main/java/org/apache/hadoop/mapred/YarnTezDagChild.java b/tez-dag/src/main/java/org/apache/hadoop/mapred/YarnTezDagChild.java
index a3f2427..3846234 100644
--- a/tez-dag/src/main/java/org/apache/hadoop/mapred/YarnTezDagChild.java
+++ b/tez-dag/src/main/java/org/apache/hadoop/mapred/YarnTezDagChild.java
@@ -420,38 +420,37 @@ public class YarnTezDagChild {
         childUGI.doAs(new PrivilegedExceptionAction<Object>() {
           @Override
           public Object run() throws Exception {
-            LOG.info("Initializing task"
-                + ", taskAttemptId=" + currentTaskAttemptID);
-            currentTask.initialize();
-            if (!currentTask.hadFatalError()) {
-              LOG.info("Running task"
-                  + ", taskAttemptId=" + currentTaskAttemptID);
-              currentTask.run();
-              LOG.info("Closing task"
+            try {
+              LOG.info("Initializing task"
                   + ", taskAttemptId=" + currentTaskAttemptID);
-              currentTask.close();
-            }
-            LOG.info("Task completed"
-                + ", taskAttemptId=" + currentTaskAttemptID
-                + ", fatalErrorOccurred=" + currentTask.hadFatalError());
-            // TODONEWTEZ check if task had a fatal error before
-            // sending completed event
-            if (!currentTask.hadFatalError()) {
-              TezEvent statusUpdateEvent =
-                  new TezEvent(new TaskStatusUpdateEvent(
-                      currentTask.getCounters(), currentTask.getProgress()),
-                      new EventMetaData(EventProducerConsumerType.SYSTEM,
-                          currentTask.getVertexName(), "",
-                          currentTask.getTaskAttemptID()));
-              TezEvent taskCompletedEvent =
-                  new TezEvent(new TaskAttemptCompletedEvent(), sourceInfo);
-              heartbeat(Arrays.asList(statusUpdateEvent, taskCompletedEvent));
+              currentTask.initialize();
+              if (!currentTask.hadFatalError()) {
+                LOG.info("Running task"
+                    + ", taskAttemptId=" + currentTaskAttemptID);
+                currentTask.run();
+                LOG.info("Closing task"
+                    + ", taskAttemptId=" + currentTaskAttemptID);
+                currentTask.close();
+              }
+              LOG.info("Task completed"
+                  + ", taskAttemptId=" + currentTaskAttemptID
+                  + ", fatalErrorOccurred=" + currentTask.hadFatalError());
+              if (!currentTask.hadFatalError()) {
+                TezEvent statusUpdateEvent =
+                    new TezEvent(new TaskStatusUpdateEvent(
+                        currentTask.getCounters(), currentTask.getProgress()),
+                        new EventMetaData(EventProducerConsumerType.SYSTEM,
+                            currentTask.getVertexName(), "",
+                            currentTask.getTaskAttemptID()));
+                TezEvent taskCompletedEvent =
+                    new TezEvent(new TaskAttemptCompletedEvent(), sourceInfo);
+                heartbeat(Arrays.asList(statusUpdateEvent, taskCompletedEvent));
+              }
+            } finally {
+              currentTask.cleanup();
             }
             try {
               taskLock.writeLock().lock();
-              if (currentTask != null) {
-                currentTask.cleanup();
-              }
               currentTask = null;
               currentTaskAttemptID = null;
             } finally {