|
@@ -18,50 +18,66 @@
|
|
|
|
|
|
package org.apache.hadoop.yarn.server.resourcemanager.scheduler.fair;
|
|
|
|
|
|
+import static org.junit.Assert.assertEquals;
|
|
|
+import static org.junit.Assert.assertFalse;
|
|
|
import static org.junit.Assert.assertTrue;
|
|
|
import static org.mockito.Mockito.mock;
|
|
|
import static org.mockito.Mockito.when;
|
|
|
|
|
|
+import java.io.File;
|
|
|
+import java.io.FileWriter;
|
|
|
import java.io.IOException;
|
|
|
+import java.io.PrintWriter;
|
|
|
+import java.util.Collection;
|
|
|
|
|
|
-import org.apache.hadoop.conf.Configuration;
|
|
|
import org.apache.hadoop.yarn.api.records.Resource;
|
|
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
|
-import org.apache.hadoop.yarn.event.AsyncDispatcher;
|
|
|
-import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.MockNodes;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.MockRM;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.NodeAddedSchedulerEvent;
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.NodeUpdateSchedulerEvent;
|
|
|
import org.apache.hadoop.yarn.util.resource.Resources;
|
|
|
+import org.junit.After;
|
|
|
import org.junit.Before;
|
|
|
import org.junit.Test;
|
|
|
import org.mockito.Mockito;
|
|
|
|
|
|
-public class TestFSLeafQueue {
|
|
|
- private FSLeafQueue schedulable = null;
|
|
|
- private Resource maxResource = Resources.createResource(10);
|
|
|
+public class TestFSLeafQueue extends FairSchedulerTestBase {
|
|
|
+ private final static String ALLOC_FILE = new File(TEST_DIR,
|
|
|
+ TestFSLeafQueue.class.getName() + ".xml").getAbsolutePath();
|
|
|
+ private Resource maxResource = Resources.createResource(1024 * 8);
|
|
|
|
|
|
@Before
|
|
|
public void setup() throws IOException {
|
|
|
- FairScheduler scheduler = new FairScheduler();
|
|
|
- Configuration conf = createConfiguration();
|
|
|
- // All tests assume only one assignment per node update
|
|
|
- conf.set(FairSchedulerConfiguration.ASSIGN_MULTIPLE, "false");
|
|
|
- ResourceManager resourceManager = new ResourceManager();
|
|
|
- resourceManager.init(conf);
|
|
|
- ((AsyncDispatcher)resourceManager.getRMContext().getDispatcher()).start();
|
|
|
- scheduler.init(conf);
|
|
|
- scheduler.start();
|
|
|
- scheduler.reinitialize(conf, resourceManager.getRMContext());
|
|
|
-
|
|
|
- String queueName = "root.queue1";
|
|
|
- scheduler.allocConf = mock(AllocationConfiguration.class);
|
|
|
- when(scheduler.allocConf.getMaxResources(queueName)).thenReturn(maxResource);
|
|
|
- when(scheduler.allocConf.getMinResources(queueName)).thenReturn(Resources.none());
|
|
|
+ conf = createConfiguration();
|
|
|
+ conf.setClass(YarnConfiguration.RM_SCHEDULER, FairScheduler.class,
|
|
|
+ ResourceScheduler.class);
|
|
|
+ }
|
|
|
|
|
|
- schedulable = new FSLeafQueue(queueName, scheduler, null);
|
|
|
+ @After
|
|
|
+ public void teardown() {
|
|
|
+ if (resourceManager != null) {
|
|
|
+ resourceManager.stop();
|
|
|
+ resourceManager = null;
|
|
|
+ }
|
|
|
+ conf = null;
|
|
|
}
|
|
|
|
|
|
@Test
|
|
|
public void testUpdateDemand() {
|
|
|
+ conf.set(FairSchedulerConfiguration.ASSIGN_MULTIPLE, "false");
|
|
|
+ resourceManager = new MockRM(conf);
|
|
|
+ resourceManager.start();
|
|
|
+ scheduler = (FairScheduler) resourceManager.getResourceScheduler();
|
|
|
+ scheduler.allocConf = mock(AllocationConfiguration.class);
|
|
|
+
|
|
|
+ String queueName = "root.queue1";
|
|
|
+ when(scheduler.allocConf.getMaxResources(queueName)).thenReturn(maxResource);
|
|
|
+ when(scheduler.allocConf.getMinResources(queueName)).thenReturn(Resources.none());
|
|
|
+ FSLeafQueue schedulable = new FSLeafQueue(queueName, scheduler, null);
|
|
|
+
|
|
|
FSAppAttempt app = mock(FSAppAttempt.class);
|
|
|
Mockito.when(app.getDemand()).thenReturn(maxResource);
|
|
|
|
|
@@ -73,11 +89,137 @@ public class TestFSLeafQueue {
|
|
|
assertTrue("Demand is greater than max allowed ",
|
|
|
Resources.equals(schedulable.getDemand(), maxResource));
|
|
|
}
|
|
|
-
|
|
|
- private Configuration createConfiguration() {
|
|
|
- Configuration conf = new YarnConfiguration();
|
|
|
- conf.setClass(YarnConfiguration.RM_SCHEDULER, FairScheduler.class,
|
|
|
- ResourceScheduler.class);
|
|
|
- return conf;
|
|
|
+
|
|
|
+ @Test (timeout = 5000)
|
|
|
+ public void test() throws Exception {
|
|
|
+ conf.set(FairSchedulerConfiguration.ALLOCATION_FILE, ALLOC_FILE);
|
|
|
+ PrintWriter out = new PrintWriter(new FileWriter(ALLOC_FILE));
|
|
|
+ out.println("<?xml version=\"1.0\"?>");
|
|
|
+ out.println("<allocations>");
|
|
|
+ out.println("<queue name=\"queueA\">");
|
|
|
+ out.println("<minResources>2048mb,0vcores</minResources>");
|
|
|
+ out.println("</queue>");
|
|
|
+ out.println("<queue name=\"queueB\">");
|
|
|
+ out.println("<minResources>2048mb,0vcores</minResources>");
|
|
|
+ out.println("</queue>");
|
|
|
+ out.println("</allocations>");
|
|
|
+ out.close();
|
|
|
+
|
|
|
+ resourceManager = new MockRM(conf);
|
|
|
+ resourceManager.start();
|
|
|
+ scheduler = (FairScheduler) resourceManager.getResourceScheduler();
|
|
|
+
|
|
|
+ // Add one big node (only care about aggregate capacity)
|
|
|
+ RMNode node1 =
|
|
|
+ MockNodes.newNodeInfo(1, Resources.createResource(4 * 1024, 4), 1,
|
|
|
+ "127.0.0.1");
|
|
|
+ NodeAddedSchedulerEvent nodeEvent1 = new NodeAddedSchedulerEvent(node1);
|
|
|
+ scheduler.handle(nodeEvent1);
|
|
|
+
|
|
|
+ scheduler.update();
|
|
|
+
|
|
|
+ // Queue A wants 3 * 1024. Node update gives this all to A
|
|
|
+ createSchedulingRequest(3 * 1024, "queueA", "user1");
|
|
|
+ scheduler.update();
|
|
|
+ NodeUpdateSchedulerEvent nodeEvent2 = new NodeUpdateSchedulerEvent(node1);
|
|
|
+ scheduler.handle(nodeEvent2);
|
|
|
+
|
|
|
+ // Queue B arrives and wants 1 * 1024
|
|
|
+ createSchedulingRequest(1 * 1024, "queueB", "user1");
|
|
|
+ scheduler.update();
|
|
|
+ Collection<FSLeafQueue> queues = scheduler.getQueueManager().getLeafQueues();
|
|
|
+ assertEquals(3, queues.size());
|
|
|
+
|
|
|
+ // Queue A should be above min share, B below.
|
|
|
+ FSLeafQueue queueA =
|
|
|
+ scheduler.getQueueManager().getLeafQueue("queueA", false);
|
|
|
+ FSLeafQueue queueB =
|
|
|
+ scheduler.getQueueManager().getLeafQueue("queueB", false);
|
|
|
+ assertFalse(queueA.isStarvedForMinShare());
|
|
|
+ assertTrue(queueB.isStarvedForMinShare());
|
|
|
+
|
|
|
+ // Node checks in again, should allocate for B
|
|
|
+ scheduler.handle(nodeEvent2);
|
|
|
+ // Now B should have min share ( = demand here)
|
|
|
+ assertFalse(queueB.isStarvedForMinShare());
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test (timeout = 5000)
|
|
|
+ public void testIsStarvedForFairShare() throws Exception {
|
|
|
+ conf.set(FairSchedulerConfiguration.ALLOCATION_FILE, ALLOC_FILE);
|
|
|
+ PrintWriter out = new PrintWriter(new FileWriter(ALLOC_FILE));
|
|
|
+ out.println("<?xml version=\"1.0\"?>");
|
|
|
+ out.println("<allocations>");
|
|
|
+ out.println("<queue name=\"queueA\">");
|
|
|
+ out.println("<weight>.2</weight>");
|
|
|
+ out.println("</queue>");
|
|
|
+ out.println("<queue name=\"queueB\">");
|
|
|
+ out.println("<weight>.8</weight>");
|
|
|
+ out.println("<fairSharePreemptionThreshold>.4</fairSharePreemptionThreshold>");
|
|
|
+ out.println("<queue name=\"queueB1\">");
|
|
|
+ out.println("</queue>");
|
|
|
+ out.println("<queue name=\"queueB2\">");
|
|
|
+ out.println("<fairSharePreemptionThreshold>.6</fairSharePreemptionThreshold>");
|
|
|
+ out.println("</queue>");
|
|
|
+ out.println("</queue>");
|
|
|
+ out.println("<defaultFairSharePreemptionThreshold>.5</defaultFairSharePreemptionThreshold>");
|
|
|
+ out.println("</allocations>");
|
|
|
+ out.close();
|
|
|
+
|
|
|
+ resourceManager = new MockRM(conf);
|
|
|
+ resourceManager.start();
|
|
|
+ scheduler = (FairScheduler) resourceManager.getResourceScheduler();
|
|
|
+
|
|
|
+ // Add one big node (only care about aggregate capacity)
|
|
|
+ RMNode node1 =
|
|
|
+ MockNodes.newNodeInfo(1, Resources.createResource(10 * 1024, 10), 1,
|
|
|
+ "127.0.0.1");
|
|
|
+ NodeAddedSchedulerEvent nodeEvent1 = new NodeAddedSchedulerEvent(node1);
|
|
|
+ scheduler.handle(nodeEvent1);
|
|
|
+
|
|
|
+ scheduler.update();
|
|
|
+
|
|
|
+ // Queue A wants 4 * 1024. Node update gives this all to A
|
|
|
+ createSchedulingRequest(1 * 1024, "queueA", "user1", 4);
|
|
|
+ scheduler.update();
|
|
|
+ NodeUpdateSchedulerEvent nodeEvent2 = new NodeUpdateSchedulerEvent(node1);
|
|
|
+ for (int i = 0; i < 4; i ++) {
|
|
|
+ scheduler.handle(nodeEvent2);
|
|
|
+ }
|
|
|
+
|
|
|
+ QueueManager queueMgr = scheduler.getQueueManager();
|
|
|
+ FSLeafQueue queueA = queueMgr.getLeafQueue("queueA", false);
|
|
|
+ assertEquals(4 * 1024, queueA.getResourceUsage().getMemory());
|
|
|
+
|
|
|
+ // Both queue B1 and queue B2 want 3 * 1024
|
|
|
+ createSchedulingRequest(1 * 1024, "queueB.queueB1", "user1", 3);
|
|
|
+ createSchedulingRequest(1 * 1024, "queueB.queueB2", "user1", 3);
|
|
|
+ scheduler.update();
|
|
|
+ for (int i = 0; i < 4; i ++) {
|
|
|
+ scheduler.handle(nodeEvent2);
|
|
|
+ }
|
|
|
+
|
|
|
+ FSLeafQueue queueB1 = queueMgr.getLeafQueue("queueB.queueB1", false);
|
|
|
+ FSLeafQueue queueB2 = queueMgr.getLeafQueue("queueB.queueB2", false);
|
|
|
+ assertEquals(2 * 1024, queueB1.getResourceUsage().getMemory());
|
|
|
+ assertEquals(2 * 1024, queueB2.getResourceUsage().getMemory());
|
|
|
+
|
|
|
+ // For queue B1, the fairSharePreemptionThreshold is 0.4, and the fair share
|
|
|
+ // threshold is 1.6 * 1024
|
|
|
+ assertFalse(queueB1.isStarvedForFairShare());
|
|
|
+
|
|
|
+ // For queue B2, the fairSharePreemptionThreshold is 0.6, and the fair share
|
|
|
+ // threshold is 2.4 * 1024
|
|
|
+ assertTrue(queueB2.isStarvedForFairShare());
|
|
|
+
|
|
|
+ // Node checks in again
|
|
|
+ scheduler.handle(nodeEvent2);
|
|
|
+ scheduler.handle(nodeEvent2);
|
|
|
+ assertEquals(3 * 1024, queueB1.getResourceUsage().getMemory());
|
|
|
+ assertEquals(3 * 1024, queueB2.getResourceUsage().getMemory());
|
|
|
+
|
|
|
+ // Both queue B1 and queue B2 usages go to 3 * 1024
|
|
|
+ assertFalse(queueB1.isStarvedForFairShare());
|
|
|
+ assertFalse(queueB2.isStarvedForFairShare());
|
|
|
}
|
|
|
}
|