|
@@ -19,22 +19,29 @@
|
|
|
package org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity;
|
|
|
|
|
|
import java.util.ArrayList;
|
|
|
+import java.util.HashSet;
|
|
|
+import java.util.Map;
|
|
|
import java.util.Set;
|
|
|
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
import org.apache.hadoop.yarn.api.records.ApplicationResourceUsageReport;
|
|
|
+import org.apache.hadoop.yarn.api.records.ContainerExitStatus;
|
|
|
import org.apache.hadoop.yarn.api.records.ContainerId;
|
|
|
+import org.apache.hadoop.yarn.api.records.ContainerState;
|
|
|
+import org.apache.hadoop.yarn.api.records.ContainerStatus;
|
|
|
import org.apache.hadoop.yarn.api.records.NodeId;
|
|
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
|
-import org.apache.hadoop.yarn.nodelabels.CommonNodeLabelsManager;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.MockAM;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.MockNM;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.MockRM;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.nodelabels.NullRMNodeLabelsManager;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.nodelabels.RMNodeLabelsManager;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainerEventType;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainerState;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.common.fica.FiCaSchedulerApp;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.NodeLabelsUpdateSchedulerEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.webapp.dao.ResourceInfo;
|
|
|
import org.junit.Assert;
|
|
|
import org.junit.Before;
|
|
@@ -97,8 +104,18 @@ public class TestCapacitySchedulerNodeLabelUpdate {
|
|
|
.getMemory());
|
|
|
}
|
|
|
|
|
|
+ private void checkUserUsedResource(MockRM rm, String queueName,
|
|
|
+ String userName, String partition, int memory) {
|
|
|
+ CapacityScheduler scheduler = (CapacityScheduler) rm.getResourceScheduler();
|
|
|
+ LeafQueue queue = (LeafQueue) scheduler.getQueue(queueName);
|
|
|
+ LeafQueue.User user = queue.getUser(userName);
|
|
|
+ Assert.assertEquals(memory,
|
|
|
+ user.getResourceUsage().getUsed(partition).getMemory());
|
|
|
+ }
|
|
|
+
|
|
|
@Test(timeout = 60000)
|
|
|
- public void testResourceUsage() throws Exception {
|
|
|
+ public void testRequestContainerAfterNodePartitionUpdated()
|
|
|
+ throws Exception {
|
|
|
// set node -> label
|
|
|
mgr.addToCluserNodeLabelsWithDefaultExclusivity(ImmutableSet.of("x", "y",
|
|
|
"z"));
|
|
@@ -160,7 +177,8 @@ public class TestCapacitySchedulerNodeLabelUpdate {
|
|
|
}
|
|
|
|
|
|
@Test (timeout = 60000)
|
|
|
- public void testNodeUpdate() throws Exception {
|
|
|
+ public void testResourceUsageWhenNodeUpdatesPartition()
|
|
|
+ throws Exception {
|
|
|
// set node -> label
|
|
|
mgr.addToCluserNodeLabelsWithDefaultExclusivity(ImmutableSet.of("x", "y", "z"));
|
|
|
|
|
@@ -183,8 +201,9 @@ public class TestCapacitySchedulerNodeLabelUpdate {
|
|
|
MockNM nm1 = rm.registerNode("h1:1234", 8000);
|
|
|
MockNM nm2 = rm.registerNode("h2:1234", 8000);
|
|
|
MockNM nm3 = rm.registerNode("h3:1234", 8000);
|
|
|
-
|
|
|
- ContainerId containerId;
|
|
|
+
|
|
|
+ ContainerId containerId1;
|
|
|
+ ContainerId containerId2;
|
|
|
|
|
|
// launch an app to queue a1 (label = x), and check all container will
|
|
|
// be allocated in h1
|
|
@@ -193,9 +212,9 @@ public class TestCapacitySchedulerNodeLabelUpdate {
|
|
|
|
|
|
// request a container.
|
|
|
am1.allocate("*", GB, 1, new ArrayList<ContainerId>(), "x");
|
|
|
- containerId =
|
|
|
- ContainerId.newContainerId(am1.getApplicationAttemptId(), 2);
|
|
|
- Assert.assertTrue(rm.waitForState(nm1, containerId,
|
|
|
+ containerId1 = ContainerId.newContainerId(am1.getApplicationAttemptId(), 1);
|
|
|
+ containerId2 = ContainerId.newContainerId(am1.getApplicationAttemptId(), 2);
|
|
|
+ Assert.assertTrue(rm.waitForState(nm1, containerId2,
|
|
|
RMContainerState.ALLOCATED, 10 * 1000));
|
|
|
|
|
|
// check used resource:
|
|
@@ -203,55 +222,205 @@ public class TestCapacitySchedulerNodeLabelUpdate {
|
|
|
checkUsedResource(rm, "a", 1024, "x");
|
|
|
checkUsedResource(rm, "a", 1024);
|
|
|
|
|
|
- // change h1's label to z, container should be killed
|
|
|
- mgr.replaceLabelsOnNode(ImmutableMap.of(NodeId.newInstance("h1", 0),
|
|
|
- toSet("z")));
|
|
|
- Assert.assertTrue(rm.waitForState(nm1, containerId,
|
|
|
- RMContainerState.KILLED, 10 * 1000));
|
|
|
+ CapacityScheduler cs = (CapacityScheduler) rm.getResourceScheduler();
|
|
|
+ FiCaSchedulerApp app = cs.getApplicationAttempt(am1.getApplicationAttemptId());
|
|
|
|
|
|
- // check used resource:
|
|
|
- // queue-a used x=0G, ""=1G ("" not changed)
|
|
|
+ // change h1's label to z
|
|
|
+ cs.handle(new NodeLabelsUpdateSchedulerEvent(ImmutableMap.of(nm1.getNodeId(),
|
|
|
+ toSet("z"))));
|
|
|
checkUsedResource(rm, "a", 0, "x");
|
|
|
+ checkUsedResource(rm, "a", 1024, "z");
|
|
|
checkUsedResource(rm, "a", 1024);
|
|
|
+ checkUsedResource(rm, "root", 0, "x");
|
|
|
+ checkUsedResource(rm, "root", 1024, "z");
|
|
|
+ checkUsedResource(rm, "root", 1024);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "x", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "z", 1024);
|
|
|
+ Assert.assertEquals(0,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("x").getMemory());
|
|
|
+ Assert.assertEquals(1024,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("z").getMemory());
|
|
|
|
|
|
- // request a container with label = y
|
|
|
- am1.allocate("*", GB, 1, new ArrayList<ContainerId>(), "y");
|
|
|
- containerId =
|
|
|
- ContainerId.newContainerId(am1.getApplicationAttemptId(), 3);
|
|
|
- Assert.assertTrue(rm.waitForState(nm2, containerId,
|
|
|
- RMContainerState.ALLOCATED, 10 * 1000));
|
|
|
-
|
|
|
- // check used resource:
|
|
|
- // queue-a used y=1G, ""=1G
|
|
|
+ // change h1's label to y
|
|
|
+ cs.handle(new NodeLabelsUpdateSchedulerEvent(ImmutableMap.of(nm1.getNodeId(),
|
|
|
+ toSet("y"))));
|
|
|
+ checkUsedResource(rm, "a", 0, "x");
|
|
|
checkUsedResource(rm, "a", 1024, "y");
|
|
|
+ checkUsedResource(rm, "a", 0, "z");
|
|
|
checkUsedResource(rm, "a", 1024);
|
|
|
+ checkUsedResource(rm, "root", 0, "x");
|
|
|
+ checkUsedResource(rm, "root", 1024, "y");
|
|
|
+ checkUsedResource(rm, "root", 0, "z");
|
|
|
+ checkUsedResource(rm, "root", 1024);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "x", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "y", 1024);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "z", 0);
|
|
|
+ Assert.assertEquals(0,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("x").getMemory());
|
|
|
+ Assert.assertEquals(1024,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("y").getMemory());
|
|
|
+ Assert.assertEquals(0,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("z").getMemory());
|
|
|
|
|
|
- // change h2's label to no label, container should be killed
|
|
|
- mgr.replaceLabelsOnNode(ImmutableMap.of(NodeId.newInstance("h2", 0),
|
|
|
- CommonNodeLabelsManager.EMPTY_STRING_SET));
|
|
|
- Assert.assertTrue(rm.waitForState(nm1, containerId,
|
|
|
- RMContainerState.KILLED, 10 * 1000));
|
|
|
+ // change h1's label to no label
|
|
|
+ Set<String> emptyLabels = new HashSet<>();
|
|
|
+ Map<NodeId,Set<String>> map = ImmutableMap.of(nm1.getNodeId(),
|
|
|
+ emptyLabels);
|
|
|
+ cs.handle(new NodeLabelsUpdateSchedulerEvent(map));
|
|
|
+ checkUsedResource(rm, "a", 0, "x");
|
|
|
+ checkUsedResource(rm, "a", 0, "y");
|
|
|
+ checkUsedResource(rm, "a", 0, "z");
|
|
|
+ checkUsedResource(rm, "a", 2048);
|
|
|
+ checkUsedResource(rm, "root", 0, "x");
|
|
|
+ checkUsedResource(rm, "root", 0, "y");
|
|
|
+ checkUsedResource(rm, "root", 0, "z");
|
|
|
+ checkUsedResource(rm, "root", 2048);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "x", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "y", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "z", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "", 2048);
|
|
|
+ Assert.assertEquals(0,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("x").getMemory());
|
|
|
+ Assert.assertEquals(0,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("y").getMemory());
|
|
|
+ Assert.assertEquals(0,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("z").getMemory());
|
|
|
+ Assert.assertEquals(2048,
|
|
|
+ app.getAppAttemptResourceUsage().getUsed("").getMemory());
|
|
|
+
|
|
|
+ // Finish the two containers, we should see used resource becomes 0
|
|
|
+ cs.completedContainer(cs.getRMContainer(containerId2),
|
|
|
+ ContainerStatus.newInstance(containerId2, ContainerState.COMPLETE, "",
|
|
|
+ ContainerExitStatus.KILLED_BY_RESOURCEMANAGER),
|
|
|
+ RMContainerEventType.KILL);
|
|
|
+ cs.completedContainer(cs.getRMContainer(containerId1),
|
|
|
+ ContainerStatus.newInstance(containerId1, ContainerState.COMPLETE, "",
|
|
|
+ ContainerExitStatus.KILLED_BY_RESOURCEMANAGER),
|
|
|
+ RMContainerEventType.KILL);
|
|
|
|
|
|
- // check used resource:
|
|
|
- // queue-a used x=0G, y=0G, ""=1G ("" not changed)
|
|
|
checkUsedResource(rm, "a", 0, "x");
|
|
|
checkUsedResource(rm, "a", 0, "y");
|
|
|
- checkUsedResource(rm, "a", 1024);
|
|
|
+ checkUsedResource(rm, "a", 0, "z");
|
|
|
+ checkUsedResource(rm, "a", 0);
|
|
|
+ checkUsedResource(rm, "root", 0, "x");
|
|
|
+ checkUsedResource(rm, "root", 0, "y");
|
|
|
+ checkUsedResource(rm, "root", 0, "z");
|
|
|
+ checkUsedResource(rm, "root", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "x", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "y", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "z", 0);
|
|
|
+ checkUserUsedResource(rm, "a", "user", "", 0);
|
|
|
+
|
|
|
+ rm.close();
|
|
|
+ }
|
|
|
+
|
|
|
+
|
|
|
+ @Test (timeout = 60000)
|
|
|
+ public void testComplexResourceUsageWhenNodeUpdatesPartition()
|
|
|
+ throws Exception {
|
|
|
+ /*
|
|
|
+ * This test is similar to testResourceUsageWhenNodeUpdatesPartition, this
|
|
|
+ * will include multiple applications, multiple users and multiple
|
|
|
+ * containers running on a single node, size of each container is 1G
|
|
|
+ *
|
|
|
+ * Node 1
|
|
|
+ * ------
|
|
|
+ * App1-container3
|
|
|
+ * App2-container2
|
|
|
+ * App2-Container3
|
|
|
+ *
|
|
|
+ * Node 2
|
|
|
+ * ------
|
|
|
+ * App2-container1
|
|
|
+ * App1-container1
|
|
|
+ * App1-container2
|
|
|
+ */
|
|
|
+ // set node -> label
|
|
|
+ mgr.addToCluserNodeLabelsWithDefaultExclusivity(ImmutableSet.of("x", "y", "z"));
|
|
|
|
|
|
+ // set mapping:
|
|
|
+ // h1 -> x
|
|
|
+ // h2 -> y
|
|
|
+ mgr.addLabelsToNode(ImmutableMap.of(NodeId.newInstance("h1", 0), toSet("x")));
|
|
|
+
|
|
|
+ // inject node label manager
|
|
|
+ MockRM rm = new MockRM(getConfigurationWithQueueLabels(conf)) {
|
|
|
+ @Override
|
|
|
+ public RMNodeLabelsManager createNodeLabelManager() {
|
|
|
+ return mgr;
|
|
|
+ }
|
|
|
+ };
|
|
|
+
|
|
|
+ rm.getRMContext().setNodeLabelManager(mgr);
|
|
|
+ rm.start();
|
|
|
+ MockNM nm1 = rm.registerNode("h1:1234", 80000);
|
|
|
+ MockNM nm2 = rm.registerNode("h2:1234", 80000);
|
|
|
+
|
|
|
+ // app1
|
|
|
+ RMApp app1 = rm.submitApp(GB, "app", "u1", null, "a");
|
|
|
+ MockAM am1 = MockRM.launchAndRegisterAM(app1, rm, nm2);
|
|
|
+
|
|
|
+ // c2 on n1, c3 on n2
|
|
|
+ am1.allocate("*", GB, 1, new ArrayList<ContainerId>(), "x");
|
|
|
+ ContainerId containerId =
|
|
|
+ ContainerId.newContainerId(am1.getApplicationAttemptId(), 2);
|
|
|
+ Assert.assertTrue(rm.waitForState(nm1, containerId,
|
|
|
+ RMContainerState.ALLOCATED, 10 * 1000));
|
|
|
+ am1.allocate("*", GB, 1, new ArrayList<ContainerId>());
|
|
|
containerId =
|
|
|
- ContainerId.newContainerId(am1.getApplicationAttemptId(), 1);
|
|
|
+ ContainerId.newContainerId(am1.getApplicationAttemptId(), 3);
|
|
|
+ Assert.assertTrue(rm.waitForState(nm2, containerId,
|
|
|
+ RMContainerState.ALLOCATED, 10 * 1000));
|
|
|
|
|
|
- // change h3's label to z, AM container should be killed
|
|
|
- mgr.replaceLabelsOnNode(ImmutableMap.of(NodeId.newInstance("h3", 0),
|
|
|
- toSet("z")));
|
|
|
+ // app2
|
|
|
+ RMApp app2 = rm.submitApp(GB, "app", "u2", null, "a");
|
|
|
+ MockAM am2 = MockRM.launchAndRegisterAM(app2, rm, nm2);
|
|
|
+
|
|
|
+ // c2/c3 on n1
|
|
|
+ am2.allocate("*", GB, 2, new ArrayList<ContainerId>(), "x");
|
|
|
+ containerId =
|
|
|
+ ContainerId.newContainerId(am2.getApplicationAttemptId(), 3);
|
|
|
Assert.assertTrue(rm.waitForState(nm1, containerId,
|
|
|
- RMContainerState.KILLED, 10 * 1000));
|
|
|
+ RMContainerState.ALLOCATED, 10 * 1000));
|
|
|
|
|
|
// check used resource:
|
|
|
- // queue-a used x=0G, y=0G, ""=1G ("" not changed)
|
|
|
+ // queue-a used x=1G, ""=1G
|
|
|
+ checkUsedResource(rm, "a", 3 * GB, "x");
|
|
|
+ checkUsedResource(rm, "a", 3 * GB);
|
|
|
+
|
|
|
+ CapacityScheduler cs = (CapacityScheduler) rm.getResourceScheduler();
|
|
|
+ FiCaSchedulerApp application1 =
|
|
|
+ cs.getApplicationAttempt(am1.getApplicationAttemptId());
|
|
|
+ FiCaSchedulerApp application2 =
|
|
|
+ cs.getApplicationAttempt(am2.getApplicationAttemptId());
|
|
|
+
|
|
|
+ // change h1's label to z
|
|
|
+ cs.handle(new NodeLabelsUpdateSchedulerEvent(ImmutableMap.of(nm1.getNodeId(),
|
|
|
+ toSet("z"))));
|
|
|
checkUsedResource(rm, "a", 0, "x");
|
|
|
- checkUsedResource(rm, "a", 0, "y");
|
|
|
- checkUsedResource(rm, "a", 0);
|
|
|
+ checkUsedResource(rm, "a", 3 * GB, "z");
|
|
|
+ checkUsedResource(rm, "a", 3 * GB);
|
|
|
+ checkUsedResource(rm, "root", 0, "x");
|
|
|
+ checkUsedResource(rm, "root", 3 * GB, "z");
|
|
|
+ checkUsedResource(rm, "root", 3 * GB);
|
|
|
+ checkUserUsedResource(rm, "a", "u1", "x", 0 * GB);
|
|
|
+ checkUserUsedResource(rm, "a", "u1", "z", 1 * GB);
|
|
|
+ checkUserUsedResource(rm, "a", "u1", "", 2 * GB);
|
|
|
+ checkUserUsedResource(rm, "a", "u2", "x", 0 * GB);
|
|
|
+ checkUserUsedResource(rm, "a", "u2", "z", 2 * GB);
|
|
|
+ checkUserUsedResource(rm, "a", "u2", "", 1 * GB);
|
|
|
+ Assert.assertEquals(0,
|
|
|
+ application1.getAppAttemptResourceUsage().getUsed("x").getMemory());
|
|
|
+ Assert.assertEquals(1 * GB,
|
|
|
+ application1.getAppAttemptResourceUsage().getUsed("z").getMemory());
|
|
|
+ Assert.assertEquals(2 * GB,
|
|
|
+ application1.getAppAttemptResourceUsage().getUsed("").getMemory());
|
|
|
+ Assert.assertEquals(0,
|
|
|
+ application2.getAppAttemptResourceUsage().getUsed("x").getMemory());
|
|
|
+ Assert.assertEquals(2 * GB,
|
|
|
+ application2.getAppAttemptResourceUsage().getUsed("z").getMemory());
|
|
|
+ Assert.assertEquals(1 * GB,
|
|
|
+ application2.getAppAttemptResourceUsage().getUsed("").getMemory());
|
|
|
|
|
|
rm.close();
|
|
|
}
|