Bläddra i källkod

YARN-3248. Display count of nodes blacklisted by apps in the web UI.
Contributed by Varun Vasudev

(cherry picked from commit 4728bdfa15809db4b8b235faa286c65de4a48cf6)
(cherry picked from commit e26b6e55e96b763063dfbd39977096367eafc1e3)

Xuan 10 år sedan
förälder
incheckning
7af5d6b4ba
14 ändrade filer med 476 tillägg och 86 borttagningar
  1. 3 0
      hadoop-yarn-project/CHANGES.txt
  2. 37 29
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/webapp/AppAttemptBlock.java
  3. 25 13
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/webapp/AppBlock.java
  4. 56 39
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/webapp/AppsBlock.java
  5. 4 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/scheduler/AppSchedulingInfo.java
  6. 4 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/scheduler/SchedulerApplicationAttempt.java
  7. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/AppsBlockWithMetrics.java
  8. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/CapacitySchedulerPage.java
  9. 68 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/RMAppAttemptBlock.java
  10. 110 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/RMAppBlock.java
  11. 146 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/RMAppsBlock.java
  12. 2 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/RMWebServices.java
  13. 18 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/dao/AppAttemptInfo.java
  14. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/test/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/TestRMWebServicesApps.java

+ 3 - 0
hadoop-yarn-project/CHANGES.txt

@@ -217,6 +217,9 @@ Release 2.6.1 - 2015-09-09
     YARN-3379. Fixed missing data in localityTable and ResourceRequests table
     in RM WebUI. (Xuan Gong via jianhe)
 
+    YARN-3248. Display count of nodes blacklisted by apps in the web UI.
+    (Varun Vasudev via xgong)
+
 Release 2.6.0 - 2014-11-18
 
   INCOMPATIBLE CHANGES

+ 37 - 29
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/webapp/AppAttemptBlock.java

@@ -43,6 +43,7 @@ import org.apache.hadoop.yarn.webapp.hamlet.Hamlet.TABLE;
 import org.apache.hadoop.yarn.webapp.hamlet.Hamlet.TBODY;
 import org.apache.hadoop.yarn.webapp.view.HtmlBlock;
 import org.apache.hadoop.yarn.webapp.view.InfoBlock;
+
 import com.google.inject.Inject;
 
 public class AppAttemptBlock extends HtmlBlock {
@@ -73,7 +74,7 @@ public class AppAttemptBlock extends HtmlBlock {
     }
 
     UserGroupInformation callerUGI = getCallerUGI();
-    ApplicationAttemptReport appAttemptReport = null;
+    ApplicationAttemptReport appAttemptReport;
     try {
       final GetApplicationAttemptReportRequest request =
           GetApplicationAttemptReportRequest.newInstance(appAttemptId);
@@ -136,33 +137,7 @@ public class AppAttemptBlock extends HtmlBlock {
         && appAttempt.getRpcPort() < 65536) {
       node = appAttempt.getHost() + ":" + appAttempt.getRpcPort();
     }
-    info("Application Attempt Overview")
-      ._(
-        "Application Attempt State:",
-        appAttempt.getAppAttemptState() == null ? UNAVAILABLE : appAttempt
-          .getAppAttemptState())
-      ._(
-        "AM Container:",
-        appAttempt.getAmContainerId() == null || containers == null
-            || !hasAMContainer(appAttemptReport.getAMContainerId(), containers)
-            ? null : root_url("container", appAttempt.getAmContainerId()),
-        String.valueOf(appAttempt.getAmContainerId()))
-      ._("Node:", node)
-      ._(
-        "Tracking URL:",
-        appAttempt.getTrackingUrl() == null
-            || appAttempt.getTrackingUrl() == UNAVAILABLE ? null
-            : root_url(appAttempt.getTrackingUrl()),
-        appAttempt.getTrackingUrl() == null
-            || appAttempt.getTrackingUrl() == UNAVAILABLE
-            ? "Unassigned"
-            : appAttempt.getAppAttemptState() == YarnApplicationAttemptState.FINISHED
-                || appAttempt.getAppAttemptState() == YarnApplicationAttemptState.FAILED
-                || appAttempt.getAppAttemptState() == YarnApplicationAttemptState.KILLED
-                ? "History" : "ApplicationMaster")
-      ._("Diagnostics Info:", appAttempt.getDiagnosticsInfo() == null ?
-          "" : appAttempt.getDiagnosticsInfo());
-
+    generateOverview(appAttemptReport, containers, appAttempt, node);
     html._(InfoBlock.class);
 
     if (exceptionWhenGetContainerReports) {
@@ -216,7 +191,40 @@ public class AppAttemptBlock extends HtmlBlock {
     tbody._()._();
   }
 
-  private boolean hasAMContainer(ContainerId containerId,
+  protected void generateOverview(ApplicationAttemptReport appAttemptReport,
+      Collection<ContainerReport> containers, AppAttemptInfo appAttempt,
+      String node) {
+    info("Application Attempt Overview")
+      ._(
+        "Application Attempt State:",
+        appAttempt.getAppAttemptState() == null ? UNAVAILABLE : appAttempt
+          .getAppAttemptState())
+      ._(
+        "AM Container:",
+        appAttempt.getAmContainerId() == null || containers == null
+            || !hasAMContainer(appAttemptReport.getAMContainerId(), containers)
+            ? null : root_url("container", appAttempt.getAmContainerId()),
+        String.valueOf(appAttempt.getAmContainerId()))
+      ._("Node:", node)
+      ._(
+        "Tracking URL:",
+        appAttempt.getTrackingUrl() == null
+            || appAttempt.getTrackingUrl().equals(UNAVAILABLE) ? null
+            : root_url(appAttempt.getTrackingUrl()),
+        appAttempt.getTrackingUrl() == null
+            || appAttempt.getTrackingUrl().equals(UNAVAILABLE)
+            ? "Unassigned"
+            : appAttempt.getAppAttemptState() == YarnApplicationAttemptState.FINISHED
+                || appAttempt.getAppAttemptState() == YarnApplicationAttemptState.FAILED
+                || appAttempt.getAppAttemptState() == YarnApplicationAttemptState.KILLED
+                ? "History" : "ApplicationMaster")
+      ._(
+        "Diagnostics Info:",
+        appAttempt.getDiagnosticsInfo() == null ? "" : appAttempt
+          .getDiagnosticsInfo());
+  }
+
+  protected boolean hasAMContainer(ContainerId containerId,
       Collection<ContainerReport> containers) {
     for (ContainerReport container : containers) {
       if (containerId.equals(container.getContainerId())) {

+ 25 - 13
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/webapp/AppBlock.java

@@ -24,6 +24,7 @@ import static org.apache.hadoop.yarn.webapp.YarnWebParams.WEB_UI_TYPE;
 
 import java.security.PrivilegedExceptionAction;
 import java.util.Collection;
+
 import org.apache.commons.lang.StringEscapeUtils;
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
@@ -89,7 +90,7 @@ public class AppBlock extends HtmlBlock {
     }
 
     UserGroupInformation callerUGI = getCallerUGI();
-    ApplicationReport appReport = null;
+    ApplicationReport appReport;
     try {
       final GetApplicationReportRequest request =
           GetApplicationReportRequest.newInstance(appID);
@@ -161,7 +162,8 @@ public class AppBlock extends HtmlBlock {
       ._("Application Type:", app.getType())
       ._("Application Tags:",
         app.getApplicationTags() == null ? "" : app.getApplicationTags())
-      ._("YarnApplicationState:",
+      ._(
+        "YarnApplicationState:",
         app.getAppState() == null ? UNAVAILABLE : clarifyAppState(app
           .getAppState()))
       ._("FinalStatus Reported by AM:",
@@ -171,16 +173,19 @@ public class AppBlock extends HtmlBlock {
         "Elapsed:",
         StringUtils.formatTime(Times.elapsed(app.getStartedTime(),
           app.getFinishedTime())))
-      ._("Tracking URL:",
-        app.getTrackingUrl() == null || app.getTrackingUrl() == UNAVAILABLE
-            ? null : root_url(app.getTrackingUrl()),
-        app.getTrackingUrl() == null || app.getTrackingUrl() == UNAVAILABLE
-            ? "Unassigned" : app.getAppState() == YarnApplicationState.FINISHED
-                || app.getAppState() == YarnApplicationState.FAILED
-                || app.getAppState() == YarnApplicationState.KILLED ? "History"
-                : "ApplicationMaster")
+      ._(
+        "Tracking URL:",
+        app.getTrackingUrl() == null
+            || app.getTrackingUrl().equals(UNAVAILABLE) ? null : root_url(app
+          .getTrackingUrl()),
+        app.getTrackingUrl() == null
+            || app.getTrackingUrl().equals(UNAVAILABLE) ? "Unassigned" : app
+          .getAppState() == YarnApplicationState.FINISHED
+            || app.getAppState() == YarnApplicationState.FAILED
+            || app.getAppState() == YarnApplicationState.KILLED ? "History"
+            : "ApplicationMaster")
       ._("Diagnostics:",
-          app.getDiagnosticsInfo() == null ? "" : app.getDiagnosticsInfo());
+        app.getDiagnosticsInfo() == null ? "" : app.getDiagnosticsInfo());
 
     Collection<ApplicationAttemptReport> attempts;
     try {
@@ -211,6 +216,13 @@ public class AppBlock extends HtmlBlock {
 
     html._(InfoBlock.class);
 
+    generateApplicationTable(html, callerUGI, attempts);
+
+  }
+
+  protected void generateApplicationTable(Block html,
+      UserGroupInformation callerUGI,
+      Collection<ApplicationAttemptReport> attempts) {
     // Application Attempt Table
     TBODY<TABLE<Hamlet>> tbody =
         html.table("#attempts").thead().tr().th(".id", "Attempt ID")
@@ -220,7 +232,7 @@ public class AppBlock extends HtmlBlock {
     StringBuilder attemptsTableData = new StringBuilder("[\n");
     for (final ApplicationAttemptReport appAttemptReport : attempts) {
       AppAttemptInfo appAttempt = new AppAttemptInfo(appAttemptReport);
-      ContainerReport containerReport = null;
+      ContainerReport containerReport;
       try {
         // AM container is always the first container of the attempt
         final GetContainerReportRequest request =
@@ -231,7 +243,7 @@ public class AppBlock extends HtmlBlock {
               appBaseProt.getContainerReport(request).getContainerReport();
         } else {
           containerReport = callerUGI.doAs(
-              new PrivilegedExceptionAction<ContainerReport> () {
+              new PrivilegedExceptionAction<ContainerReport>() {
             @Override
             public ContainerReport run() throws Exception {
               ContainerReport report = null;

+ 56 - 39
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/webapp/AppsBlock.java

@@ -24,6 +24,7 @@ import static org.apache.hadoop.yarn.webapp.YarnWebParams.APPS_NUM;
 import static org.apache.hadoop.yarn.webapp.view.JQueryUI.C_PROGRESSBAR;
 import static org.apache.hadoop.yarn.webapp.view.JQueryUI.C_PROGRESSBAR_VALUE;
 
+import java.io.IOException;
 import java.security.PrivilegedExceptionAction;
 import java.util.Collection;
 import java.util.EnumSet;
@@ -36,6 +37,7 @@ import org.apache.hadoop.yarn.api.ApplicationBaseProtocol;
 import org.apache.hadoop.yarn.api.protocolrecords.GetApplicationsRequest;
 import org.apache.hadoop.yarn.api.records.ApplicationReport;
 import org.apache.hadoop.yarn.api.records.YarnApplicationState;
+import org.apache.hadoop.yarn.exceptions.YarnException;
 import org.apache.hadoop.yarn.server.webapp.dao.AppInfo;
 import org.apache.hadoop.yarn.webapp.hamlet.Hamlet;
 import org.apache.hadoop.yarn.webapp.hamlet.Hamlet.TABLE;
@@ -48,26 +50,19 @@ public class AppsBlock extends HtmlBlock {
 
   private static final Log LOG = LogFactory.getLog(AppsBlock.class);
   protected ApplicationBaseProtocol appBaseProt;
+  protected EnumSet<YarnApplicationState> reqAppStates;
+  protected UserGroupInformation callerUGI;
+  protected Collection<ApplicationReport> appReports;
 
   @Inject
-  AppsBlock(ApplicationBaseProtocol appBaseProt, ViewContext ctx) {
+  protected AppsBlock(ApplicationBaseProtocol appBaseProt, ViewContext ctx) {
     super(ctx);
     this.appBaseProt = appBaseProt;
   }
 
-  @Override
-  public void render(Block html) {
-    setTitle("Applications");
-
-    TBODY<TABLE<Hamlet>> tbody =
-        html.table("#apps").thead().tr().th(".id", "ID").th(".user", "User")
-          .th(".name", "Name").th(".type", "Application Type")
-          .th(".queue", "Queue").th(".starttime", "StartTime")
-          .th(".finishtime", "FinishTime").th(".state", "State")
-          .th(".finalstatus", "FinalStatus").th(".progress", "Progress")
-          .th(".ui", "Tracking UI")._()._().tbody();
-    EnumSet<YarnApplicationState> reqAppStates =
-        EnumSet.noneOf(YarnApplicationState.class);
+  protected void fetchData() throws YarnException, IOException,
+      InterruptedException {
+    reqAppStates = EnumSet.noneOf(YarnApplicationState.class);
     String reqStateString = $(APP_STATE);
     if (reqStateString != null && !reqStateString.isEmpty()) {
       String[] appStateStrings = reqStateString.split(",");
@@ -76,33 +71,54 @@ public class AppsBlock extends HtmlBlock {
       }
     }
 
-    UserGroupInformation callerUGI = getCallerUGI();
-    Collection<ApplicationReport> appReports = null;
+    callerUGI = getCallerUGI();
+    final GetApplicationsRequest request =
+        GetApplicationsRequest.newInstance(reqAppStates);
+    String appsNumStr = $(APPS_NUM);
+    if (appsNumStr != null && !appsNumStr.isEmpty()) {
+      long appsNum = Long.parseLong(appsNumStr);
+      request.setLimit(appsNum);
+    }
+    if (callerUGI == null) {
+      appReports = appBaseProt.getApplications(request).getApplicationList();
+    } else {
+      appReports =
+          callerUGI
+            .doAs(new PrivilegedExceptionAction<Collection<ApplicationReport>>() {
+              @Override
+              public Collection<ApplicationReport> run() throws Exception {
+                return appBaseProt.getApplications(request)
+                  .getApplicationList();
+              }
+            });
+    }
+  }
+
+  @Override
+  public void render(Block html) {
+    setTitle("Applications");
+
     try {
-      final GetApplicationsRequest request =
-          GetApplicationsRequest.newInstance(reqAppStates);
-      String appsNumStr = $(APPS_NUM);
-      if (appsNumStr != null && !appsNumStr.isEmpty()) {
-        long appsNum = Long.parseLong(appsNumStr);
-        request.setLimit(appsNum);
-      }
-      if (callerUGI == null) {
-        appReports = appBaseProt.getApplications(request).getApplicationList();
-      } else {
-        appReports = callerUGI.doAs(
-            new PrivilegedExceptionAction<Collection<ApplicationReport>> () {
-          @Override
-          public Collection<ApplicationReport> run() throws Exception {
-            return appBaseProt.getApplications(request).getApplicationList();
-          }
-        });
-      }
-    } catch (Exception e) {
+      fetchData();
+    }
+    catch( Exception e) {
       String message = "Failed to read the applications.";
       LOG.error(message, e);
       html.p()._(message)._();
       return;
     }
+    renderData(html);
+  }
+
+  protected void renderData(Block html) {
+    TBODY<TABLE<Hamlet>> tbody =
+        html.table("#apps").thead().tr().th(".id", "ID").th(".user", "User")
+          .th(".name", "Name").th(".type", "Application Type")
+          .th(".queue", "Queue").th(".starttime", "StartTime")
+          .th(".finishtime", "FinishTime").th(".state", "State")
+          .th(".finalstatus", "FinalStatus").th(".progress", "Progress")
+          .th(".ui", "Tracking UI")._()._().tbody();
+
     StringBuilder appsTableData = new StringBuilder("[\n");
     for (ApplicationReport appReport : appReports) {
       // TODO: remove the following condition. It is still here because
@@ -123,7 +139,7 @@ public class AppsBlock extends HtmlBlock {
         .append("</a>\",\"")
         .append(
           StringEscapeUtils.escapeJavaScript(StringEscapeUtils.escapeHtml(app
-            .getUser())))
+              .getUser())))
         .append("\",\"")
         .append(
           StringEscapeUtils.escapeJavaScript(StringEscapeUtils.escapeHtml(app
@@ -150,11 +166,12 @@ public class AppsBlock extends HtmlBlock {
         .append("'> </div> </div>").append("\",\"<a ");
 
       String trackingURL =
-          app.getTrackingUrl() == null || app.getTrackingUrl() == UNAVAILABLE
-              ? null : app.getTrackingUrl();
+          app.getTrackingUrl() == null
+              || app.getTrackingUrl().equals(UNAVAILABLE) ? null : app
+            .getTrackingUrl();
 
       String trackingUI =
-          app.getTrackingUrl() == null || app.getTrackingUrl() == UNAVAILABLE
+          app.getTrackingUrl() == null || app.getTrackingUrl().equals(UNAVAILABLE)
               ? "Unassigned"
               : app.getAppState() == YarnApplicationState.FINISHED
                   || app.getAppState() == YarnApplicationState.FAILED

+ 4 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/scheduler/AppSchedulingInfo.java

@@ -437,6 +437,10 @@ public class AppSchedulingInfo {
     return this.blacklist;
   }
 
+  public synchronized Set<String> getBlackListCopy() {
+    return new HashSet<String>(this.blacklist);
+  }
+
   public synchronized void transferStateFromPreviousAppSchedulingInfo(
       AppSchedulingInfo appInfo) {
     //    this.priorities = appInfo.getPriorities();

+ 4 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/scheduler/SchedulerApplicationAttempt.java

@@ -615,4 +615,8 @@ public class SchedulerApplicationAttempt {
     // schedulingOpportunities
     // lastScheduledContainer
   }
+
+  public Set<String> getBlacklistedNodes() {
+    return this.appSchedulingInfo.getBlackListCopy();
+  }
 }

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/AppsBlockWithMetrics.java

@@ -27,6 +27,6 @@ import org.apache.hadoop.yarn.webapp.view.HtmlBlock;
 class AppsBlockWithMetrics extends HtmlBlock {
   @Override public void render(Block html) {
     html._(MetricsOverviewTable.class);
-    html._(AppsBlock.class);
+    html._(RMAppsBlock.class);
   }
 }

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/CapacitySchedulerPage.java

@@ -230,7 +230,7 @@ class CapacitySchedulerPage extends RmView {
       ul._()._().
       script().$type("text/javascript").
           _("$('#cs').hide();")._()._().
-      _(AppsBlock.class);
+      _(RMAppsBlock.class);
     }
   }
 

+ 68 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/RMAppAttemptBlock.java

@@ -18,12 +18,23 @@
 
 package org.apache.hadoop.yarn.server.resourcemanager.webapp;
 
+import java.util.Collection;
+import java.util.Set;
+
+import org.apache.commons.lang.StringUtils;
 import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
+import org.apache.hadoop.yarn.api.records.ApplicationAttemptReport;
 import org.apache.hadoop.yarn.api.records.ApplicationId;
+import org.apache.hadoop.yarn.api.records.ContainerReport;
+import org.apache.hadoop.yarn.api.records.YarnApplicationAttemptState;
 import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
 import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
 import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.AbstractYarnScheduler;
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.SchedulerApplicationAttempt;
 import org.apache.hadoop.yarn.server.webapp.AppAttemptBlock;
+import org.apache.hadoop.yarn.server.webapp.dao.AppAttemptInfo;
 
 import com.google.inject.Inject;
 
@@ -53,4 +64,61 @@ public class RMAppAttemptBlock extends AppAttemptBlock{
     }
     return attempt;
   }
+
+  protected void generateOverview(ApplicationAttemptReport appAttemptReport,
+      Collection<ContainerReport> containers, AppAttemptInfo appAttempt,
+      String node) {
+
+    String blacklistedNodes = "-";
+    Set<String> nodes =
+        getBlacklistedNodes(rm, getRMAppAttempt().getAppAttemptId());
+    if (nodes != null) {
+      if (!nodes.isEmpty()) {
+        blacklistedNodes = StringUtils.join(nodes, ", ");
+      }
+    }
+
+    info("Application Attempt Overview")
+      ._(
+        "Application Attempt State:",
+        appAttempt.getAppAttemptState() == null ? UNAVAILABLE : appAttempt
+          .getAppAttemptState())
+      ._(
+        "AM Container:",
+        appAttempt.getAmContainerId() == null || containers == null
+            || !hasAMContainer(appAttemptReport.getAMContainerId(), containers)
+            ? null : root_url("container", appAttempt.getAmContainerId()),
+        String.valueOf(appAttempt.getAmContainerId()))
+      ._("Node:", node)
+      ._(
+        "Tracking URL:",
+        appAttempt.getTrackingUrl() == null
+            || appAttempt.getTrackingUrl().equals(UNAVAILABLE) ? null
+            : root_url(appAttempt.getTrackingUrl()),
+        appAttempt.getTrackingUrl() == null
+            || appAttempt.getTrackingUrl().equals(UNAVAILABLE)
+            ? "Unassigned"
+            : appAttempt.getAppAttemptState() == YarnApplicationAttemptState.FINISHED
+                || appAttempt.getAppAttemptState() == YarnApplicationAttemptState.FAILED
+                || appAttempt.getAppAttemptState() == YarnApplicationAttemptState.KILLED
+                ? "History" : "ApplicationMaster")
+      ._(
+        "Diagnostics Info:",
+        appAttempt.getDiagnosticsInfo() == null ? "" : appAttempt
+          .getDiagnosticsInfo())._("Blacklisted Nodes:", blacklistedNodes);
+  }
+
+  public static Set<String> getBlacklistedNodes(ResourceManager rm,
+      ApplicationAttemptId appid) {
+    if (rm.getResourceScheduler() instanceof AbstractYarnScheduler) {
+      AbstractYarnScheduler ayScheduler =
+          (AbstractYarnScheduler) rm.getResourceScheduler();
+      SchedulerApplicationAttempt attempt =
+          ayScheduler.getApplicationAttempt(appid);
+      if (attempt != null) {
+        return attempt.getBlacklistedNodes();
+      }
+    }
+    return null;
+  }
 }

+ 110 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/RMAppBlock.java

@@ -20,13 +20,29 @@ package org.apache.hadoop.yarn.server.resourcemanager.webapp;
 
 import static org.apache.hadoop.yarn.webapp.view.JQueryUI._INFO_WRAP;
 
+import java.security.PrivilegedExceptionAction;
+import java.util.Collection;
+import java.util.Set;
+
+import org.apache.commons.lang.StringEscapeUtils;
+import org.apache.commons.logging.Log;
+import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.security.UserGroupInformation;
+import org.apache.hadoop.yarn.api.protocolrecords.GetContainerReportRequest;
+import org.apache.hadoop.yarn.api.records.ApplicationAttemptReport;
+import org.apache.hadoop.yarn.api.records.ContainerId;
+import org.apache.hadoop.yarn.api.records.ContainerReport;
 import org.apache.hadoop.yarn.api.records.Resource;
+import org.apache.hadoop.yarn.exceptions.ContainerNotFoundException;
 import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
 import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
 import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppMetrics;
 import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptMetrics;
 import org.apache.hadoop.yarn.server.webapp.AppBlock;
+import org.apache.hadoop.yarn.server.webapp.dao.AppAttemptInfo;
+import org.apache.hadoop.yarn.server.webapp.dao.ContainerInfo;
+import org.apache.hadoop.yarn.util.ConverterUtils;
 import org.apache.hadoop.yarn.util.resource.Resources;
 import org.apache.hadoop.yarn.webapp.hamlet.Hamlet;
 import org.apache.hadoop.yarn.webapp.hamlet.Hamlet.DIV;
@@ -36,8 +52,10 @@ import com.google.inject.Inject;
 
 public class RMAppBlock extends AppBlock{
 
+  private static final Log LOG = LogFactory.getLog(RMAppBlock.class);
   private final ResourceManager rm;
 
+
   @Inject
   RMAppBlock(ViewContext ctx, Configuration conf, ResourceManager rm) {
     super(rm.getClientRMService(), ctx, conf);
@@ -91,4 +109,96 @@ public class RMAppBlock extends AppBlock{
               appMetrics == null ? "N/A" : appMetrics.getVcoreSeconds()));
     pdiv._();
   }
+
+  @Override
+  protected void generateApplicationTable(Block html,
+      UserGroupInformation callerUGI,
+      Collection<ApplicationAttemptReport> attempts) {
+    // Application Attempt Table
+    Hamlet.TBODY<Hamlet.TABLE<Hamlet>> tbody =
+        html.table("#attempts").thead().tr().th(".id", "Attempt ID")
+            .th(".started", "Started").th(".node", "Node").th(".logs", "Logs")
+            .th(".blacklistednodes", "Blacklisted Nodes")._()._().tbody();
+
+    StringBuilder attemptsTableData = new StringBuilder("[\n");
+    for (final ApplicationAttemptReport appAttemptReport : attempts) {
+      AppAttemptInfo appAttempt = new AppAttemptInfo(appAttemptReport);
+      ContainerReport containerReport = null;
+      try {
+        // AM container is always the first container of the attempt
+        final GetContainerReportRequest request =
+            GetContainerReportRequest.newInstance(ContainerId.newContainerId(
+                appAttemptReport.getApplicationAttemptId(), 1));
+        if (callerUGI == null) {
+          containerReport =
+              appBaseProt.getContainerReport(request).getContainerReport();
+        } else {
+          containerReport = callerUGI.doAs(
+              new PrivilegedExceptionAction<ContainerReport>() {
+                @Override
+                public ContainerReport run() throws Exception {
+                  ContainerReport report = null;
+                  try {
+                    report = appBaseProt.getContainerReport(request)
+                        .getContainerReport();
+                  } catch (ContainerNotFoundException ex) {
+                    LOG.warn(ex.getMessage());
+                  }
+                  return report;
+                }
+              });
+        }
+      } catch (Exception e) {
+        String message =
+            "Failed to read the AM container of the application attempt "
+                + appAttemptReport.getApplicationAttemptId() + ".";
+        LOG.error(message, e);
+        html.p()._(message)._();
+        return;
+      }
+      long startTime = 0L;
+      String logsLink = null;
+      String nodeLink = null;
+      if (containerReport != null) {
+        ContainerInfo container = new ContainerInfo(containerReport);
+        startTime = container.getStartedTime();
+        logsLink = containerReport.getLogUrl();
+        nodeLink = containerReport.getNodeHttpAddress();
+      }
+      String blacklistedNodesCount = "N/A";
+      Set<String> nodes = RMAppAttemptBlock.getBlacklistedNodes(rm,
+          ConverterUtils.toApplicationAttemptId(appAttempt.getAppAttemptId()));
+      if(nodes != null) {
+        blacklistedNodesCount = String.valueOf(nodes.size());
+      }
+
+      // AppAttemptID numerical value parsed by parseHadoopID in
+      // yarn.dt.plugins.js
+      attemptsTableData
+          .append("[\"<a href='")
+          .append(url("appattempt", appAttempt.getAppAttemptId()))
+          .append("'>")
+          .append(appAttempt.getAppAttemptId())
+          .append("</a>\",\"")
+          .append(startTime)
+          .append("\",\"<a ")
+          .append(nodeLink == null ? "#" : "href='" + nodeLink)
+          .append("'>")
+          .append(nodeLink == null ? "N/A" : StringEscapeUtils
+              .escapeJavaScript(StringEscapeUtils.escapeHtml(nodeLink)))
+          .append("</a>\",\"<a ")
+          .append(logsLink == null ? "#" : "href='" + logsLink).append("'>")
+          .append(logsLink == null ? "N/A" : "Logs").append("</a>\",").append(
+          "\"").append(blacklistedNodesCount).append("\"],\n");
+    }
+    if (attemptsTableData.charAt(attemptsTableData.length() - 2) == ',') {
+      attemptsTableData.delete(attemptsTableData.length() - 2,
+          attemptsTableData.length() - 1);
+    }
+    attemptsTableData.append("]");
+    html.script().$type("text/javascript")
+        ._("var attemptsTableData=" + attemptsTableData)._();
+
+    tbody._()._();
+  }
 }

+ 146 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/RMAppsBlock.java

@@ -0,0 +1,146 @@
+/**
+ * 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.hadoop.yarn.server.resourcemanager.webapp;
+
+import static org.apache.hadoop.yarn.util.StringHelper.join;
+import static org.apache.hadoop.yarn.webapp.view.JQueryUI.C_PROGRESSBAR;
+import static org.apache.hadoop.yarn.webapp.view.JQueryUI.C_PROGRESSBAR_VALUE;
+
+import java.util.Set;
+
+import org.apache.commons.lang.StringEscapeUtils;
+import org.apache.hadoop.yarn.api.ApplicationBaseProtocol;
+import org.apache.hadoop.yarn.api.records.ApplicationReport;
+import org.apache.hadoop.yarn.api.records.YarnApplicationState;
+import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
+import org.apache.hadoop.yarn.server.webapp.AppsBlock;
+import org.apache.hadoop.yarn.server.webapp.dao.AppInfo;
+import org.apache.hadoop.yarn.util.ConverterUtils;
+import org.apache.hadoop.yarn.webapp.View;
+import org.apache.hadoop.yarn.webapp.hamlet.Hamlet;
+import org.apache.hadoop.yarn.webapp.hamlet.Hamlet.TABLE;
+import org.apache.hadoop.yarn.webapp.hamlet.Hamlet.TBODY;
+
+import com.google.inject.Inject;
+
+public class RMAppsBlock extends AppsBlock {
+
+  private ResourceManager rm;
+
+  @Inject
+  RMAppsBlock(ResourceManager rm, ApplicationBaseProtocol appBaseProt,
+      View.ViewContext ctx) {
+    super(appBaseProt, ctx);
+    this.rm = rm;
+  }
+
+  @Override
+  protected void renderData(Block html) {
+    TBODY<TABLE<Hamlet>> tbody =
+        html.table("#apps").thead().tr().th(".id", "ID").th(".user", "User")
+          .th(".name", "Name").th(".type", "Application Type")
+          .th(".queue", "Queue").th(".starttime", "StartTime")
+          .th(".finishtime", "FinishTime").th(".state", "State")
+          .th(".finalstatus", "FinalStatus").th(".progress", "Progress")
+          .th(".ui", "Tracking UI").th(".blacklisted", "Blacklisted Nodes")._()
+          ._().tbody();
+
+    StringBuilder appsTableData = new StringBuilder("[\n");
+    for (ApplicationReport appReport : appReports) {
+      // TODO: remove the following condition. It is still here because
+      // the history side implementation of ApplicationBaseProtocol
+      // hasn't filtering capability (YARN-1819).
+      if (!reqAppStates.isEmpty()
+          && !reqAppStates.contains(appReport.getYarnApplicationState())) {
+        continue;
+      }
+
+      AppInfo app = new AppInfo(appReport);
+      String blacklistedNodesCount = "N/A";
+      Set<String> nodes =
+          RMAppAttemptBlock
+            .getBlacklistedNodes(rm, ConverterUtils.toApplicationAttemptId(app
+              .getCurrentAppAttemptId()));
+      if (nodes != null) {
+        blacklistedNodesCount = String.valueOf(nodes.size());
+      }
+      String percent = String.format("%.1f", app.getProgress());
+      // AppID numerical value parsed by parseHadoopID in yarn.dt.plugins.js
+      appsTableData
+        .append("[\"<a href='")
+        .append(url("app", app.getAppId()))
+        .append("'>")
+        .append(app.getAppId())
+        .append("</a>\",\"")
+        .append(
+          StringEscapeUtils.escapeJavaScript(StringEscapeUtils.escapeHtml(app
+            .getUser())))
+        .append("\",\"")
+        .append(
+          StringEscapeUtils.escapeJavaScript(StringEscapeUtils.escapeHtml(app
+            .getName())))
+        .append("\",\"")
+        .append(
+          StringEscapeUtils.escapeJavaScript(StringEscapeUtils.escapeHtml(app
+            .getType())))
+        .append("\",\"")
+        .append(
+          StringEscapeUtils.escapeJavaScript(StringEscapeUtils.escapeHtml(app
+            .getQueue()))).append("\",\"").append(app.getStartedTime())
+        .append("\",\"").append(app.getFinishedTime())
+        .append("\",\"")
+        .append(app.getAppState() == null ? UNAVAILABLE : app.getAppState())
+        .append("\",\"")
+        .append(app.getFinalAppStatus())
+        .append("\",\"")
+        // Progress bar
+        .append("<br title='").append(percent).append("'> <div class='")
+        .append(C_PROGRESSBAR).append("' title='").append(join(percent, '%'))
+        .append("'> ").append("<div class='").append(C_PROGRESSBAR_VALUE)
+        .append("' style='").append(join("width:", percent, '%'))
+        .append("'> </div> </div>").append("\",\"<a ");
+
+      String trackingURL =
+          app.getTrackingUrl() == null
+              || app.getTrackingUrl().equals(UNAVAILABLE) ? null : app
+            .getTrackingUrl();
+
+      String trackingUI =
+          app.getTrackingUrl() == null
+              || app.getTrackingUrl().equals(UNAVAILABLE) ? "Unassigned" : app
+            .getAppState() == YarnApplicationState.FINISHED
+              || app.getAppState() == YarnApplicationState.FAILED
+              || app.getAppState() == YarnApplicationState.KILLED ? "History"
+              : "ApplicationMaster";
+      appsTableData.append(trackingURL == null ? "#" : "href='" + trackingURL)
+        .append("'>").append(trackingUI).append("</a>\",").append("\"")
+        .append(blacklistedNodesCount).append("\"],\n");
+
+    }
+    if (appsTableData.charAt(appsTableData.length() - 2) == ',') {
+      appsTableData.delete(appsTableData.length() - 2,
+        appsTableData.length() - 1);
+    }
+    appsTableData.append("]");
+    html.script().$type("text/javascript")
+      ._("var appsTableData=" + appsTableData)._();
+
+    tbody._()._();
+  }
+}

+ 2 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/RMWebServices.java

@@ -631,7 +631,8 @@ public class RMWebServices {
 
     AppAttemptsInfo appAttemptsInfo = new AppAttemptsInfo();
     for (RMAppAttempt attempt : app.getAppAttempts().values()) {
-      AppAttemptInfo attemptInfo = new AppAttemptInfo(attempt, app.getUser());
+      AppAttemptInfo attemptInfo =
+          new AppAttemptInfo(rm, attempt, app.getUser());
       appAttemptsInfo.add(attemptInfo);
     }
 

+ 18 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/dao/AppAttemptInfo.java

@@ -21,8 +21,13 @@ import javax.xml.bind.annotation.XmlAccessType;
 import javax.xml.bind.annotation.XmlAccessorType;
 import javax.xml.bind.annotation.XmlRootElement;
 
+import org.apache.commons.lang.StringUtils;
 import org.apache.hadoop.yarn.api.records.Container;
+import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
 import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.AbstractYarnScheduler;
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.SchedulerApplicationAttempt;
+import org.apache.hadoop.yarn.server.resourcemanager.webapp.RMAppAttemptBlock;
 import org.apache.hadoop.yarn.util.ConverterUtils;
 import org.apache.hadoop.yarn.webapp.util.WebAppUtils;
 
@@ -36,16 +41,18 @@ public class AppAttemptInfo {
   protected String nodeHttpAddress;
   protected String nodeId;
   protected String logsLink;
+  protected String blacklistedNodes;
 
   public AppAttemptInfo() {
   }
 
-  public AppAttemptInfo(RMAppAttempt attempt, String user) {
+  public AppAttemptInfo(ResourceManager rm, RMAppAttempt attempt, String user) {
     this.startTime = 0;
     this.containerId = "";
     this.nodeHttpAddress = "";
     this.nodeId = "";
     this.logsLink = "";
+    this.blacklistedNodes = "";
     if (attempt != null) {
       this.id = attempt.getAppAttemptId().getAttemptId();
       this.startTime = attempt.getStartTime();
@@ -57,6 +64,16 @@ public class AppAttemptInfo {
         this.logsLink =
             WebAppUtils.getRunningLogURL("//" + masterContainer.getNodeHttpAddress(),
                 ConverterUtils.toString(masterContainer.getId()), user);
+        if (rm.getResourceScheduler() instanceof AbstractYarnScheduler) {
+          AbstractYarnScheduler ayScheduler =
+              (AbstractYarnScheduler) rm.getResourceScheduler();
+          SchedulerApplicationAttempt sattempt =
+              ayScheduler.getApplicationAttempt(attempt.getAppAttemptId());
+          if (sattempt != null) {
+            blacklistedNodes =
+                StringUtils.join(sattempt.getBlacklistedNodes(), ", ");
+          }
+        }
       }
     }
   }

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/test/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/TestRMWebServicesApps.java

@@ -1599,7 +1599,7 @@ public class TestRMWebServicesApps extends JerseyTest {
       String user)
       throws JSONException, Exception {
 
-    assertEquals("incorrect number of elements", 6, info.length());
+    assertEquals("incorrect number of elements", 7, info.length());
 
     verifyAppAttemptInfoGeneric(appAttempt, info.getInt("id"),
         info.getLong("startTime"), info.getString("containerId"),