Преглед на файлове

YARN-10801. Fix Auto Queue template to properly set all configuration properties. Contributed by Andras Gyori

Szilard Nemeth преди 4 години
родител
ревизия
428478bbe2

+ 15 - 3
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/scheduler/capacity/AbstractCSQueue.java

@@ -360,9 +360,8 @@ public abstract class AbstractCSQueue implements CSQueue {
 
     writeLock.lock();
     try {
-      if (isDynamicQueue() && getParent() instanceof ParentQueue) {
-        ((ParentQueue) getParent()).getAutoCreatedQueueTemplate()
-            .setTemplateEntriesForChild(configuration, getQueuePath());
+      if (isDynamicQueue()) {
+        setDynamicQueueProperties(configuration);
       }
       // get labels
       this.accessibleLabels =
@@ -478,6 +477,19 @@ public abstract class AbstractCSQueue implements CSQueue {
     }
   }
 
+  /**
+   * Set properties specific to dynamic queues.
+   * @param configuration configuration on which the properties are set
+   */
+  protected void setDynamicQueueProperties(
+      CapacitySchedulerConfiguration configuration) {
+    // Set properties from parent template
+    if (getParent() instanceof ParentQueue) {
+      ((ParentQueue) getParent()).getAutoCreatedQueueTemplate()
+          .setTemplateEntriesForChild(configuration, getQueuePath());
+    }
+  }
+
   private void setupMaximumAllocation(CapacitySchedulerConfiguration csConf) {
     String myQueuePath = getQueuePath();
     Resource clusterMax = ResourceUtils

+ 20 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/scheduler/capacity/LeafQueue.java

@@ -147,14 +147,21 @@ public class LeafQueue extends AbstractCSQueue {
   @SuppressWarnings({ "unchecked", "rawtypes" })
   public LeafQueue(CapacitySchedulerContext cs,
       String queueName, CSQueue parent, CSQueue old) throws IOException {
-      this(cs, cs.getConfiguration(), queueName, parent, old);
+    this(cs, cs.getConfiguration(), queueName, parent, old, false);
   }
 
   public LeafQueue(CapacitySchedulerContext cs,
       CapacitySchedulerConfiguration configuration,
-      String queueName, CSQueue parent, CSQueue old) throws
+      String queueName, CSQueue parent, CSQueue old) throws IOException {
+    this(cs, configuration, queueName, parent, old, false);
+  }
+
+  public LeafQueue(CapacitySchedulerContext cs,
+      CapacitySchedulerConfiguration configuration,
+      String queueName, CSQueue parent, CSQueue old, boolean isDynamic) throws
       IOException {
     super(cs, configuration, queueName, parent, old);
+    setDynamicQueue(isDynamic);
     this.scheduler = cs;
 
     this.usersManager = new UsersManager(metrics, this, labelManager, scheduler,
@@ -1691,6 +1698,17 @@ public class LeafQueue extends AbstractCSQueue {
     }
   }
 
+  @Override
+  protected void setDynamicQueueProperties(
+      CapacitySchedulerConfiguration configuration) {
+    super.setDynamicQueueProperties(configuration);
+    // set to -1, to disable it
+    configuration.setUserLimitFactor(getQueuePath(), -1);
+    // Set Max AM percentage to a higher value
+    configuration.setMaximumApplicationMasterResourcePerQueuePercent(
+        getQueuePath(), 1f);
+  }
+
   private void updateSchedulerHealthForCompletedContainer(
       RMContainer rmContainer, ContainerStatus containerStatus) {
     // Update SchedulerHealth for released / preempted container

+ 13 - 28
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/scheduler/capacity/ParentQueue.java

@@ -112,14 +112,21 @@ public class ParentQueue extends AbstractCSQueue {
 
   public ParentQueue(CapacitySchedulerContext cs,
       String queueName, CSQueue parent, CSQueue old) throws IOException {
-    this(cs, cs.getConfiguration(), queueName, parent, old);
+    this(cs, cs.getConfiguration(), queueName, parent, old, false);
+  }
+  private ParentQueue(CapacitySchedulerContext cs,
+      CapacitySchedulerConfiguration csConf, String queueName,
+      CSQueue parent,
+      CSQueue old) throws IOException {
+    this(cs, csConf, queueName, parent, old, false);
   }
 
   private ParentQueue(CapacitySchedulerContext cs,
       CapacitySchedulerConfiguration csConf, String queueName, CSQueue parent,
-      CSQueue old)
+      CSQueue old, boolean isDynamic)
       throws IOException {
     super(cs, queueName, parent, old);
+    setDynamicQueue(isDynamic);
     this.scheduler = cs;
     this.rootQueue = (parent == null);
 
@@ -476,26 +483,6 @@ public class ParentQueue extends AbstractCSQueue {
         "numContainers=" + getNumContainers();
   }
 
-  private CapacitySchedulerConfiguration getConfForAutoCreatedQueue(
-      String childQueuePath, boolean isLeaf) {
-    // Copy existing config
-    CapacitySchedulerConfiguration dupCSConfig =
-        new CapacitySchedulerConfiguration(
-        csContext.getConfiguration(), false);
-    autoCreatedQueueTemplate.setTemplateEntriesForChild(dupCSConfig,
-        childQueuePath);
-    if (isLeaf) {
-      // set to -1, to disable it
-      dupCSConfig.setUserLimitFactor(childQueuePath, -1);
-
-      // Set Max AM percentage to a higher value
-      dupCSConfig.setMaximumApplicationMasterResourcePerQueuePercent(
-          childQueuePath, 0.5f);
-    }
-
-    return dupCSConfig;
-  }
-
   private CSQueue createNewQueue(String childQueuePath, boolean isLeaf)
       throws SchedulerDynamicEditException {
     try {
@@ -504,13 +491,11 @@ public class ParentQueue extends AbstractCSQueue {
           childQueuePath.lastIndexOf(".") + 1);
 
       if (isLeaf) {
-        childQueue = new LeafQueue(csContext,
-            getConfForAutoCreatedQueue(childQueuePath, isLeaf), queueShortName,
-            this, null);
+        childQueue = new LeafQueue(csContext, csContext.getConfiguration(),
+            queueShortName, this, null, true);
       } else{
-        childQueue = new ParentQueue(csContext,
-            getConfForAutoCreatedQueue(childQueuePath, isLeaf), queueShortName,
-            this, null);
+        childQueue = new ParentQueue(csContext, csContext.getConfiguration(),
+            queueShortName, this, null, true);
       }
       childQueue.setDynamicQueue(true);
       // It should be sufficient now, we don't need to set more, because weights

+ 10 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/test/java/org/apache/hadoop/yarn/server/resourcemanager/scheduler/capacity/TestCapacitySchedulerNewQueueAutoCreation.java

@@ -147,6 +147,8 @@ public class TestCapacitySchedulerNewQueueAutoCreation
     Assert.assertEquals(1f, c.getQueueCapacities().getWeight(), 1e-6);
     Assert.assertEquals(400 * GB,
         c.getQueueResourceQuotas().getEffectiveMinResource().getMemorySize());
+    Assert.assertEquals(((LeafQueue)c).getUserLimitFactor(), -1, 1e-6);
+    Assert.assertEquals(((LeafQueue)c).getMaxAMResourcePerQueuePercent(), 1, 1e-6);
 
     // Now add another queue-d, in the same hierarchy
     createQueue("root.d-auto");
@@ -696,6 +698,14 @@ public class TestCapacitySchedulerNewQueueAutoCreation
     cs.reinitialize(csConf, mockRM.getRMContext());
     Assert.assertEquals("weight is not explicitly set", 4f,
         a2.getQueueCapacities().getWeight(), 1e-6);
+
+    csConf.setBoolean(AutoCreatedQueueTemplate.getAutoQueueTemplatePrefix(
+        "root.a") + CapacitySchedulerConfiguration
+        .AUTO_CREATE_CHILD_QUEUE_AUTO_REMOVAL_ENABLE, false);
+    cs.reinitialize(csConf, mockRM.getRMContext());
+    LeafQueue a3 = createQueue("root.a.a3");
+    Assert.assertFalse("auto queue deletion should be turned off on a3",
+        a3.isEligibleForAutoDeletion());
   }
 
   @Test