|
@@ -29,11 +29,13 @@ import java.io.IOException;
|
|
|
import java.net.InetSocketAddress;
|
|
|
import java.security.PrivilegedAction;
|
|
|
import java.util.ArrayList;
|
|
|
+import java.util.Arrays;
|
|
|
import java.util.Collections;
|
|
|
import java.util.Comparator;
|
|
|
import java.util.HashMap;
|
|
|
import java.util.List;
|
|
|
import java.util.Map;
|
|
|
+import java.util.Set;
|
|
|
import java.util.concurrent.BrokenBarrierException;
|
|
|
import java.util.concurrent.CyclicBarrier;
|
|
|
|
|
@@ -111,6 +113,7 @@ import org.apache.hadoop.yarn.server.resourcemanager.scheduler.common.fica.FiCaS
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.common.fica.FiCaSchedulerNode;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.AppAddedSchedulerEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.AppAttemptAddedSchedulerEvent;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.AppAttemptRemovedSchedulerEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.NodeAddedSchedulerEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.NodeRemovedSchedulerEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.NodeUpdateSchedulerEvent;
|
|
@@ -128,10 +131,13 @@ import org.apache.hadoop.yarn.util.resource.Resources;
|
|
|
import org.junit.After;
|
|
|
import org.junit.Assert;
|
|
|
import org.junit.Before;
|
|
|
-import org.junit.ComparisonFailure;
|
|
|
import org.junit.Test;
|
|
|
import org.mockito.Mockito;
|
|
|
|
|
|
+import com.google.common.collect.ImmutableMap;
|
|
|
+import com.google.common.collect.ImmutableSet;
|
|
|
+import com.google.common.collect.Sets;
|
|
|
+
|
|
|
|
|
|
public class TestCapacityScheduler {
|
|
|
private static final Log LOG = LogFactory.getLog(TestCapacityScheduler.class);
|
|
@@ -2557,6 +2563,165 @@ public class TestCapacityScheduler {
|
|
|
Assert.fail("Shouldn't successfully allocate containers for am2, "
|
|
|
+ "queue-a's max capacity will be violated if container allocated");
|
|
|
}
|
|
|
+
|
|
|
+ @SuppressWarnings("unchecked")
|
|
|
+ private <E> Set<E> toSet(E... elements) {
|
|
|
+ Set<E> set = Sets.newHashSet(elements);
|
|
|
+ return set;
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test
|
|
|
+ public void testQueueHierarchyPendingResourceUpdate() throws Exception {
|
|
|
+ Configuration conf =
|
|
|
+ TestUtils.getConfigurationWithQueueLabels(new Configuration(false));
|
|
|
+ conf.setBoolean(YarnConfiguration.NODE_LABELS_ENABLED, true);
|
|
|
+
|
|
|
+ final RMNodeLabelsManager mgr = new NullRMNodeLabelsManager();
|
|
|
+ mgr.init(conf);
|
|
|
+ mgr.addToCluserNodeLabels(ImmutableSet.of("x", "y"));
|
|
|
+ mgr.addLabelsToNode(ImmutableMap.of(NodeId.newInstance("h1", 0), toSet("x")));
|
|
|
+
|
|
|
+ MemoryRMStateStore memStore = new MemoryRMStateStore();
|
|
|
+ memStore.init(conf);
|
|
|
+ MockRM rm = new MockRM(conf, memStore) {
|
|
|
+ protected RMNodeLabelsManager createNodeLabelManager() {
|
|
|
+ return mgr;
|
|
|
+ }
|
|
|
+ };
|
|
|
+
|
|
|
+ rm.start();
|
|
|
+ MockNM nm1 = // label = x
|
|
|
+ new MockNM("h1:1234", 200 * GB, rm.getResourceTrackerService());
|
|
|
+ nm1.registerNode();
|
|
|
+
|
|
|
+ MockNM nm2 = // label = ""
|
|
|
+ new MockNM("h2:1234", 200 * GB, rm.getResourceTrackerService());
|
|
|
+ nm2.registerNode();
|
|
|
+
|
|
|
+ // Launch app1 in queue=a1
|
|
|
+ RMApp app1 = rm.submitApp(1 * GB, "app", "user", null, "a1");
|
|
|
+ MockAM am1 = MockRM.launchAndRegisterAM(app1, rm, nm2);
|
|
|
+
|
|
|
+ // Launch app2 in queue=b1
|
|
|
+ RMApp app2 = rm.submitApp(8 * GB, "app", "user", null, "b1");
|
|
|
+ MockAM am2 = MockRM.launchAndRegisterAM(app2, rm, nm2);
|
|
|
+
|
|
|
+ // am1 asks for 8 * 1GB container for no label
|
|
|
+ am1.allocate(Arrays.asList(ResourceRequest.newInstance(
|
|
|
+ Priority.newInstance(1), "*", Resources.createResource(1 * GB), 8)),
|
|
|
+ null);
|
|
|
+
|
|
|
+ checkPendingResource(rm, "a1", 8 * GB, null);
|
|
|
+ checkPendingResource(rm, "a", 8 * GB, null);
|
|
|
+ checkPendingResource(rm, "root", 8 * GB, null);
|
|
|
+
|
|
|
+ // am2 asks for 8 * 1GB container for no label
|
|
|
+ am2.allocate(Arrays.asList(ResourceRequest.newInstance(
|
|
|
+ Priority.newInstance(1), "*", Resources.createResource(1 * GB), 8)),
|
|
|
+ null);
|
|
|
+
|
|
|
+ checkPendingResource(rm, "a1", 8 * GB, null);
|
|
|
+ checkPendingResource(rm, "a", 8 * GB, null);
|
|
|
+ checkPendingResource(rm, "b1", 8 * GB, null);
|
|
|
+ checkPendingResource(rm, "b", 8 * GB, null);
|
|
|
+ // root = a + b
|
|
|
+ checkPendingResource(rm, "root", 16 * GB, null);
|
|
|
+
|
|
|
+ // am2 asks for 8 * 1GB container in another priority for no label
|
|
|
+ am2.allocate(Arrays.asList(ResourceRequest.newInstance(
|
|
|
+ Priority.newInstance(2), "*", Resources.createResource(1 * GB), 8)),
|
|
|
+ null);
|
|
|
+
|
|
|
+ checkPendingResource(rm, "a1", 8 * GB, null);
|
|
|
+ checkPendingResource(rm, "a", 8 * GB, null);
|
|
|
+ checkPendingResource(rm, "b1", 16 * GB, null);
|
|
|
+ checkPendingResource(rm, "b", 16 * GB, null);
|
|
|
+ // root = a + b
|
|
|
+ checkPendingResource(rm, "root", 24 * GB, null);
|
|
|
+
|
|
|
+ // am1 asks 4 GB resource instead of 8 * GB for priority=1
|
|
|
+ am1.allocate(Arrays.asList(ResourceRequest.newInstance(
|
|
|
+ Priority.newInstance(1), "*", Resources.createResource(4 * GB), 1)),
|
|
|
+ null);
|
|
|
+
|
|
|
+ checkPendingResource(rm, "a1", 4 * GB, null);
|
|
|
+ checkPendingResource(rm, "a", 4 * GB, null);
|
|
|
+ checkPendingResource(rm, "b1", 16 * GB, null);
|
|
|
+ checkPendingResource(rm, "b", 16 * GB, null);
|
|
|
+ // root = a + b
|
|
|
+ checkPendingResource(rm, "root", 20 * GB, null);
|
|
|
+
|
|
|
+ // am1 asks 8 * GB resource which label=x
|
|
|
+ am1.allocate(Arrays.asList(ResourceRequest.newInstance(
|
|
|
+ Priority.newInstance(2), "*", Resources.createResource(8 * GB), 1,
|
|
|
+ true, "x")), null);
|
|
|
+
|
|
|
+ checkPendingResource(rm, "a1", 4 * GB, null);
|
|
|
+ checkPendingResource(rm, "a", 4 * GB, null);
|
|
|
+ checkPendingResource(rm, "a1", 8 * GB, "x");
|
|
|
+ checkPendingResource(rm, "a", 8 * GB, "x");
|
|
|
+ checkPendingResource(rm, "b1", 16 * GB, null);
|
|
|
+ checkPendingResource(rm, "b", 16 * GB, null);
|
|
|
+ // root = a + b
|
|
|
+ checkPendingResource(rm, "root", 20 * GB, null);
|
|
|
+ checkPendingResource(rm, "root", 8 * GB, "x");
|
|
|
+
|
|
|
+ // some containers allocated for am1, pending resource should decrease
|
|
|
+ ContainerId containerId =
|
|
|
+ ContainerId.newContainerId(am1.getApplicationAttemptId(), 2);
|
|
|
+ Assert.assertTrue(rm.waitForState(nm1, containerId,
|
|
|
+ RMContainerState.ALLOCATED, 10 * 1000));
|
|
|
+ containerId = ContainerId.newContainerId(am1.getApplicationAttemptId(), 3);
|
|
|
+ Assert.assertTrue(rm.waitForState(nm2, containerId,
|
|
|
+ RMContainerState.ALLOCATED, 10 * 1000));
|
|
|
+
|
|
|
+ checkPendingResource(rm, "a1", 0 * GB, null);
|
|
|
+ checkPendingResource(rm, "a", 0 * GB, null);
|
|
|
+ checkPendingResource(rm, "a1", 0 * GB, "x");
|
|
|
+ checkPendingResource(rm, "a", 0 * GB, "x");
|
|
|
+ // some containers could be allocated for am2 when we allocating containers
|
|
|
+ // for am1, just check if pending resource of b1/b/root > 0
|
|
|
+ checkPendingResourceGreaterThanZero(rm, "b1", null);
|
|
|
+ checkPendingResourceGreaterThanZero(rm, "b", null);
|
|
|
+ // root = a + b
|
|
|
+ checkPendingResourceGreaterThanZero(rm, "root", null);
|
|
|
+ checkPendingResource(rm, "root", 0 * GB, "x");
|
|
|
+
|
|
|
+ // complete am2, pending resource should be 0 now
|
|
|
+ AppAttemptRemovedSchedulerEvent appRemovedEvent =
|
|
|
+ new AppAttemptRemovedSchedulerEvent(
|
|
|
+ am2.getApplicationAttemptId(), RMAppAttemptState.FINISHED, false);
|
|
|
+ rm.getResourceScheduler().handle(appRemovedEvent);
|
|
|
+
|
|
|
+ checkPendingResource(rm, "a1", 0 * GB, null);
|
|
|
+ checkPendingResource(rm, "a", 0 * GB, null);
|
|
|
+ checkPendingResource(rm, "a1", 0 * GB, "x");
|
|
|
+ checkPendingResource(rm, "a", 0 * GB, "x");
|
|
|
+ checkPendingResource(rm, "b1", 0 * GB, null);
|
|
|
+ checkPendingResource(rm, "b", 0 * GB, null);
|
|
|
+ checkPendingResource(rm, "root", 0 * GB, null);
|
|
|
+ checkPendingResource(rm, "root", 0 * GB, "x");
|
|
|
+ }
|
|
|
+
|
|
|
+ private void checkPendingResource(MockRM rm, String queueName, int memory,
|
|
|
+ String label) {
|
|
|
+ CapacityScheduler cs = (CapacityScheduler) rm.getResourceScheduler();
|
|
|
+ CSQueue queue = cs.getQueue(queueName);
|
|
|
+ Assert.assertEquals(
|
|
|
+ memory,
|
|
|
+ queue.getQueueResourceUsage()
|
|
|
+ .getPending(label == null ? RMNodeLabelsManager.NO_LABEL : label)
|
|
|
+ .getMemory());
|
|
|
+ }
|
|
|
+
|
|
|
+ private void checkPendingResourceGreaterThanZero(MockRM rm, String queueName,
|
|
|
+ String label) {
|
|
|
+ CapacityScheduler cs = (CapacityScheduler) rm.getResourceScheduler();
|
|
|
+ CSQueue queue = cs.getQueue(queueName);
|
|
|
+ Assert.assertTrue(queue.getQueueResourceUsage()
|
|
|
+ .getPending(label == null ? RMNodeLabelsManager.NO_LABEL : label)
|
|
|
+ .getMemory() > 0);
|
|
|
+ }
|
|
|
|
|
|
// Test verifies AM Used resource for LeafQueue when AM ResourceRequest is
|
|
|
// lesser than minimumAllocation
|