Browse Source

YARN-10343. Legacy RM UI should include labeled metrics for allocated, total, and reserved resources. Contributed by Eric Payne

Jonathan Hung 4 years ago
parent
commit
3eaf62726f

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

@@ -200,6 +200,10 @@ public class ResourceUsage extends AbstractResourceUsage {
     return _getAll(ResourceType.USED);
   }
 
+  public Resource getAllReserved() {
+    return _getAll(ResourceType.RESERVED);
+  }
+
   // Cache Used
   public Resource getCachedUsed() {
     return _get(NL, ResourceType.CACHED_USED);

+ 41 - 8
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/MetricsOverviewTable.java

@@ -22,6 +22,7 @@ import org.apache.hadoop.util.StringUtils;
 import org.apache.hadoop.yarn.api.records.ResourceTypeInfo;
 import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
 import org.apache.hadoop.yarn.server.resourcemanager.webapp.dao.ClusterMetricsInfo;
+import org.apache.hadoop.yarn.server.resourcemanager.webapp.dao.ResourceInfo;
 import org.apache.hadoop.yarn.server.resourcemanager.webapp.dao.SchedulerInfo;
 import org.apache.hadoop.yarn.server.resourcemanager.webapp.dao.UserMetricsInfo;
 
@@ -60,7 +61,38 @@ public class MetricsOverviewTable extends HtmlBlock {
     ClusterMetricsInfo clusterMetrics = new ClusterMetricsInfo(this.rm);
     
     DIV<Hamlet> div = html.div().$class("metrics");
-    
+
+    long usedMemoryBytes = 0;
+    long totalMemoryBytes = 0;
+    long reservedMemoryBytes = 0;
+    long usedVCores = 0;
+    long totalVCores = 0;
+    long reservedVCores = 0;
+    if (clusterMetrics.getCrossPartitionMetricsAvailable()) {
+      ResourceInfo usedAllPartitions =
+          clusterMetrics.getTotalUsedResourcesAcrossPartition();
+      ResourceInfo totalAllPartitions =
+          clusterMetrics.getTotalClusterResourcesAcrossPartition();
+      ResourceInfo reservedAllPartitions =
+          clusterMetrics.getTotalReservedResourcesAcrossPartition();
+      usedMemoryBytes = usedAllPartitions.getMemorySize() * BYTES_IN_MB;
+      totalMemoryBytes = totalAllPartitions.getMemorySize() * BYTES_IN_MB;
+      reservedMemoryBytes = reservedAllPartitions.getMemorySize() * BYTES_IN_MB;
+      usedVCores = usedAllPartitions.getvCores();
+      totalVCores = totalAllPartitions.getvCores();
+      reservedVCores = reservedAllPartitions.getvCores();
+      // getTotalUsedResourcesAcrossPartition includes reserved resources.
+      usedMemoryBytes -= reservedMemoryBytes;
+      usedVCores -= reservedVCores;
+    } else {
+      usedMemoryBytes = clusterMetrics.getAllocatedMB() * BYTES_IN_MB;
+      totalMemoryBytes = clusterMetrics.getTotalMB() * BYTES_IN_MB;
+      reservedMemoryBytes = clusterMetrics.getReservedMB() * BYTES_IN_MB;
+      usedVCores = clusterMetrics.getAllocatedVirtualCores();
+      totalVCores = clusterMetrics.getTotalVirtualCores();
+      reservedVCores = clusterMetrics.getReservedVirtualCores();
+    }
+
     div.h3("Cluster Metrics").
     table("#metricsoverview").
     thead().$class("ui-widget-header").
@@ -89,13 +121,14 @@ public class MetricsOverviewTable extends HtmlBlock {
                 clusterMetrics.getAppsFailed() + clusterMetrics.getAppsKilled()
                 )
             ).
-        td(String.valueOf(clusterMetrics.getContainersAllocated())).
-        td(StringUtils.byteDesc(clusterMetrics.getAllocatedMB() * BYTES_IN_MB)).
-        td(StringUtils.byteDesc(clusterMetrics.getTotalMB() * BYTES_IN_MB)).
-        td(StringUtils.byteDesc(clusterMetrics.getReservedMB() * BYTES_IN_MB)).
-        td(String.valueOf(clusterMetrics.getAllocatedVirtualCores())).
-        td(String.valueOf(clusterMetrics.getTotalVirtualCores())).
-        td(String.valueOf(clusterMetrics.getReservedVirtualCores())).
+        td(String.valueOf(
+            clusterMetrics.getTotalAllocatedContainersAcrossPartition())).
+        td(StringUtils.byteDesc(usedMemoryBytes)).
+        td(StringUtils.byteDesc(totalMemoryBytes)).
+        td(StringUtils.byteDesc(reservedMemoryBytes)).
+        td(String.valueOf(usedVCores)).
+        td(String.valueOf(totalVCores)).
+        td(String.valueOf(reservedVCores)).
         __().
         __().__();
 

+ 26 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/webapp/dao/ClusterMetricsInfo.java

@@ -26,6 +26,7 @@ import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
 import org.apache.hadoop.yarn.server.resourcemanager.scheduler.QueueMetrics;
 import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler;
 import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CapacityScheduler;
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.ParentQueue;
 
 @XmlRootElement(name = "clusterMetrics")
 @XmlAccessorType(XmlAccessType.FIELD)
@@ -69,6 +70,14 @@ public class ClusterMetricsInfo {
   // Total registered resources of the cluster, including all partitions
   private ResourceInfo totalClusterResourcesAcrossPartition;
 
+  // Total reserved resources of the cluster, including all partitions.
+  private ResourceInfo totalReservedResourcesAcrossPartition;
+
+  // Total allocated containers across all partitions.
+  private int totalAllocatedContainersAcrossPartition;
+
+  private boolean crossPartitionMetricsAvailable = false;
+
   public ClusterMetricsInfo() {
   } // JAXB needs this
 
@@ -115,6 +124,11 @@ public class ClusterMetricsInfo {
             cs.getRootQueue().getQueueResourceUsage().getAllUsed());
         totalClusterResourcesAcrossPartition = new ResourceInfo(
             cs.getClusterResource());
+        totalReservedResourcesAcrossPartition = new ResourceInfo(
+            cs.getRootQueue().getQueueResourceUsage().getAllReserved());
+        totalAllocatedContainersAcrossPartition =
+            ((ParentQueue) cs.getRootQueue()).getNumContainers();
+        crossPartitionMetricsAvailable = true;
       }
     } else {
       this.totalMB = availableMB + allocatedMB;
@@ -346,4 +360,16 @@ public class ClusterMetricsInfo {
   public ResourceInfo getTotalClusterResourcesAcrossPartition() {
     return totalClusterResourcesAcrossPartition;
   }
+
+  public ResourceInfo getTotalReservedResourcesAcrossPartition() {
+    return totalReservedResourcesAcrossPartition;
+  }
+
+  public int getTotalAllocatedContainersAcrossPartition() {
+    return totalAllocatedContainersAcrossPartition;
+  }
+
+  public boolean getCrossPartitionMetricsAvailable() {
+    return crossPartitionMetricsAvailable;
+  }
 }

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

@@ -474,7 +474,7 @@ public class TestRMWebServices extends JerseyTestBase {
       Exception {
     assertEquals("incorrect number of elements", 1, json.length());
     JSONObject clusterinfo = json.getJSONObject("clusterMetrics");
-    assertEquals("incorrect number of elements", 27, clusterinfo.length());
+    assertEquals("incorrect number of elements", 29, clusterinfo.length());
     verifyClusterMetrics(
         clusterinfo.getInt("appsSubmitted"), clusterinfo.getInt("appsCompleted"),
         clusterinfo.getInt("reservedMB"), clusterinfo.getInt("availableMB"),