Browse Source

YARN-9213. RM Web UI v1 does not show custom resource allocations for containers page. Contributed by Szilard Nemeth.

Sunil G 6 năm trước cách đây
mục cha
commit
ba56bc24f5

+ 55 - 11
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/webapp/ContainerBlock.java

@@ -17,28 +17,30 @@
  */
 package org.apache.hadoop.yarn.server.webapp;
 
-import static org.apache.hadoop.yarn.util.StringHelper.join;
-import static org.apache.hadoop.yarn.webapp.YarnWebParams.CONTAINER_ID;
-
-import java.io.IOException;
-import java.security.PrivilegedExceptionAction;
-
+import com.google.common.annotations.VisibleForTesting;
+import com.google.inject.Inject;
 import org.apache.hadoop.security.UserGroupInformation;
 import org.apache.hadoop.util.StringUtils;
 import org.apache.hadoop.yarn.api.ApplicationBaseProtocol;
 import org.apache.hadoop.yarn.api.protocolrecords.GetContainerReportRequest;
 import org.apache.hadoop.yarn.api.records.ContainerId;
 import org.apache.hadoop.yarn.api.records.ContainerReport;
+import org.apache.hadoop.yarn.api.records.ResourceInformation;
 import org.apache.hadoop.yarn.exceptions.YarnException;
 import org.apache.hadoop.yarn.server.webapp.dao.ContainerInfo;
 import org.apache.hadoop.yarn.util.Times;
 import org.apache.hadoop.yarn.webapp.view.HtmlBlock;
 import org.apache.hadoop.yarn.webapp.view.InfoBlock;
-
-import com.google.inject.Inject;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
+import java.io.IOException;
+import java.security.PrivilegedExceptionAction;
+import java.util.Map;
+
+import static org.apache.hadoop.yarn.util.StringHelper.join;
+import static org.apache.hadoop.yarn.webapp.YarnWebParams.CONTAINER_ID;
+
 public class ContainerBlock extends HtmlBlock {
 
   private static final Logger LOG =
@@ -117,9 +119,7 @@ public class ContainerBlock extends HtmlBlock {
         StringUtils.formatTime(Times.elapsed(container.getStartedTime(),
           container.getFinishedTime())))
       .__(
-        "Resource:",
-        container.getAllocatedMB() + " Memory, "
-            + container.getAllocatedVCores() + " VCores")
+        "Resource:", getResources(container))
       .__("Logs:", container.getLogUrl() == null ? "#" : container.getLogUrl(),
           container.getLogUrl() == null ? "N/A" : "Logs")
       .__("Diagnostics:", container.getDiagnosticsInfo() == null ?
@@ -128,6 +128,50 @@ public class ContainerBlock extends HtmlBlock {
     html.__(InfoBlock.class);
   }
 
+  /**
+   * Creates a string representation of allocated resources to a container.
+   * Memory, followed with VCores are always the first two resources of
+   * the resulted string, followed with any custom resources, if any is present.
+   */
+  @VisibleForTesting
+  String getResources(ContainerInfo container) {
+    Map<String, Long> allocatedResources = container.getAllocatedResources();
+
+    StringBuilder sb = new StringBuilder();
+    sb.append(getResourceAsString(ResourceInformation.MEMORY_URI,
+        allocatedResources.get(ResourceInformation.MEMORY_URI))).append(", ");
+    sb.append(getResourceAsString(ResourceInformation.VCORES_URI,
+        allocatedResources.get(ResourceInformation.VCORES_URI)));
+
+    if (container.hasCustomResources()) {
+      container.getAllocatedResources().forEach((key, value) -> {
+        if (!key.equals(ResourceInformation.MEMORY_URI) &&
+            !key.equals(ResourceInformation.VCORES_URI)) {
+          sb.append(", ");
+          sb.append(getResourceAsString(key, value));
+        }
+      });
+    }
+
+    return sb.toString();
+  }
+
+  private String getResourceAsString(String resourceName, long value) {
+    final String translatedResourceName;
+    switch (resourceName) {
+    case ResourceInformation.MEMORY_URI:
+      translatedResourceName = "Memory";
+      break;
+    case ResourceInformation.VCORES_URI:
+      translatedResourceName = "VCores";
+      break;
+    default:
+      translatedResourceName = resourceName;
+      break;
+    }
+    return String.valueOf(value) + " " + translatedResourceName;
+  }
+
   protected ContainerReport getContainerReport(
       final GetContainerReportRequest request)
       throws YarnException, IOException {

+ 4 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/main/java/org/apache/hadoop/yarn/server/webapp/dao/ContainerInfo.java

@@ -164,4 +164,8 @@ public class ContainerInfo {
   public String getExposedPorts() {
     return exposedPorts;
   }
+
+  public boolean hasCustomResources() {
+    return allocatedResources.size() > 2;
+  }
 }

+ 93 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-common/src/test/java/org/apache/hadoop/yarn/server/webapp/ContainerBlockTest.java

@@ -0,0 +1,93 @@
+/*
+ * 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.webapp;
+
+import com.google.common.collect.ImmutableMap;
+import org.apache.hadoop.yarn.api.ApplicationBaseProtocol;
+import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
+import org.apache.hadoop.yarn.api.records.ApplicationId;
+import org.apache.hadoop.yarn.api.records.ContainerId;
+import org.apache.hadoop.yarn.api.records.ContainerReport;
+import org.apache.hadoop.yarn.api.records.ContainerState;
+import org.apache.hadoop.yarn.api.records.NodeId;
+import org.apache.hadoop.yarn.api.records.Priority;
+import org.apache.hadoop.yarn.api.records.Resource;
+import org.apache.hadoop.yarn.api.records.ResourceInformation;
+import org.apache.hadoop.yarn.resourcetypes.ResourceTypesTestHelper;
+import org.apache.hadoop.yarn.server.webapp.dao.ContainerInfo;
+import org.apache.hadoop.yarn.util.resource.CustomResourceTypesConfigurationProvider;
+import org.apache.hadoop.yarn.webapp.View;
+import org.junit.Assert;
+import org.junit.Test;
+
+import java.util.ArrayList;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+
+import static org.apache.hadoop.yarn.conf.YarnConfiguration.DEFAULT_RM_SCHEDULER_MAXIMUM_ALLOCATION_MB;
+import static org.apache.hadoop.yarn.conf.YarnConfiguration.DEFAULT_RM_SCHEDULER_MAXIMUM_ALLOCATION_VCORES;
+import static org.mockito.Mockito.mock;
+
+/**
+ * Tests for ContainerBlock.
+ */
+public class ContainerBlockTest {
+
+  private ContainerReport createContainerReport() {
+    ApplicationId applicationId = ApplicationId.newInstance(1234, 5);
+    ApplicationAttemptId attemptId = ApplicationAttemptId.newInstance(
+        applicationId, 1);
+    ContainerId containerId = ContainerId.newContainerId(attemptId, 1);
+    Map<String, List<Map<String, String>>> ports = new HashMap<>();
+    List<Map<String, String>> list = new ArrayList<>();
+    Map<String, String> map = new HashMap<>();
+    map.put("abc", "123");
+    list.add(map);
+    ports.put("192.168.0.1", list);
+    ContainerReport container = ContainerReport.newInstance(containerId, null,
+        NodeId.newInstance("host", 1234), Priority.UNDEFINED, 1234, 5678,
+        "diagnosticInfo", "logURL", 0, ContainerState.COMPLETE,
+        "http://" + NodeId.newInstance("host", 2345).toString());
+    container.setExposedPorts(ports);
+
+    return container;
+  }
+  @Test
+  public void testRenderResourcesString() {
+    CustomResourceTypesConfigurationProvider.
+        initResourceTypes(ResourceInformation.GPU_URI);
+
+    Resource resource = ResourceTypesTestHelper.newResource(
+        DEFAULT_RM_SCHEDULER_MAXIMUM_ALLOCATION_MB,
+        DEFAULT_RM_SCHEDULER_MAXIMUM_ALLOCATION_VCORES,
+        ImmutableMap.<String, String>builder()
+            .put(ResourceInformation.GPU_URI, "5").build());
+
+    ContainerBlock block = new ContainerBlock(
+        mock(ApplicationBaseProtocol.class),
+        mock(View.ViewContext.class));
+
+    ContainerReport containerReport = createContainerReport();
+    containerReport.setAllocatedResource(resource);
+    ContainerInfo containerInfo = new ContainerInfo(containerReport);
+    String resources = block.getResources(containerInfo);
+
+    Assert.assertEquals("8192 Memory, 4 VCores, 5 yarn.io/gpu", resources);
+  }
+
+}