Browse Source

YARN-9017. PlacementRule order is not maintained in CS. Contributed by Bilwa S T.

(cherry picked from commit 35010120fbbcad8618f99abf7130e53f98879a33)
Inigo Goiri 5 years ago
parent
commit
8c8ef2f444

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

@@ -703,8 +703,11 @@ public class CapacityScheduler extends
     Set<String> distinguishRuleSet = CapacitySchedulerConfigValidator
             .validatePlacementRules(placementRuleStrs);
 
-    // add UserGroupMappingPlacementRule if absent
-    distinguishRuleSet.add(YarnConfiguration.USER_GROUP_PLACEMENT_RULE);
+    // add UserGroupMappingPlacementRule if empty,default value of
+    // yarn.scheduler.queue-placement-rules is user-group
+    if (distinguishRuleSet.isEmpty()) {
+      distinguishRuleSet.add(YarnConfiguration.USER_GROUP_PLACEMENT_RULE);
+    }
 
     placementRuleStrs = new ArrayList<>(distinguishRuleSet);
 

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

@@ -28,7 +28,7 @@ import org.slf4j.LoggerFactory;
 
 import java.io.IOException;
 import java.util.Collection;
-import java.util.HashSet;
+import java.util.LinkedHashSet;
 import java.util.Set;
 
 public final class CapacitySchedulerConfigValidator {
@@ -58,7 +58,7 @@ public final class CapacitySchedulerConfigValidator {
 
   public static Set<String> validatePlacementRules(
           Collection<String> placementRuleStrs) throws IOException {
-    Set<String> distinguishRuleSet = new HashSet<>();
+    Set<String> distinguishRuleSet = new LinkedHashSet<>();
     // fail the case if we get duplicate placementRule add in
     for (String pls : placementRuleStrs) {
       if (!distinguishRuleSet.add(pls)) {

+ 43 - 6
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/test/java/org/apache/hadoop/yarn/server/resourcemanager/placement/TestPlacementManager.java

@@ -18,7 +18,6 @@
 
 package org.apache.hadoop.yarn.server.resourcemanager.placement;
 
-import org.apache.hadoop.yarn.api.records.ApplicationId;
 import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
 import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.server.resourcemanager.MockRM;
@@ -28,7 +27,9 @@ import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler
 import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CapacityScheduler;
 import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CapacitySchedulerConfiguration;
 import org.apache.hadoop.yarn.util.Records;
+import org.junit.After;
 import org.junit.Assert;
+import org.junit.Before;
 import org.junit.Test;
 
 import java.util.ArrayList;
@@ -49,20 +50,22 @@ public class TestPlacementManager {
   public static final String PARENT_QUEUE = "c";
 
   private MockRM mockRM = null;
-
-  private static final long CLUSTER_TIMESTAMP = System.currentTimeMillis();
+  private CapacitySchedulerConfiguration conf;
 
   private String getQueueMapping(String parentQueue, String leafQueue) {
     return parentQueue + DOT + leafQueue;
   }
 
-  @Test
-  public void testPlaceApplicationWithPlacementRuleChain() throws Exception {
-    CapacitySchedulerConfiguration conf = new CapacitySchedulerConfiguration();
+  @Before
+  public void setup() {
+    conf = new CapacitySchedulerConfiguration();
     setupQueueConfiguration(conf);
     conf.setClass(YarnConfiguration.RM_SCHEDULER, CapacityScheduler.class,
         ResourceScheduler.class);
+  }
 
+  @Test
+  public void testPlaceApplicationWithPlacementRuleChain() throws Exception {
     mockRM = new MockRM(conf);
     CapacityScheduler cs = (CapacityScheduler) mockRM.getResourceScheduler();
     mockRM.start();
@@ -112,4 +115,38 @@ public class TestPlacementManager {
     }
   }
 
+  @Test
+  public void testPlacementRuleUpdationOrder() throws Exception {
+    List<QueueMapping> queueMappings = new ArrayList<>();
+    QueueMapping userQueueMapping = QueueMappingBuilder.create()
+        .type(MappingType.USER).source(USER1)
+        .queue(getQueueMapping(PARENT_QUEUE, USER1)).build();
+    UserGroupMappingPlacementRule ugRule = new UserGroupMappingPlacementRule(
+        false, Arrays.asList(userQueueMapping), null);
+
+    // Configure placement rule
+    conf.set(YarnConfiguration.QUEUE_PLACEMENT_RULES, ugRule.getName());
+    queueMappings.add(userQueueMapping);
+    conf.setQueueMappings(queueMappings);
+
+    mockRM = new MockRM(conf);
+    CapacityScheduler cs = (CapacityScheduler) mockRM.getResourceScheduler();
+    mockRM.start();
+    PlacementManager pm = cs.getRMContext().getQueuePlacementManager();
+
+    // As we are setting placement rule, It shouldn't update default
+    // placement rule ie user-group. Number of placemnt rules should be 1.
+    Assert.assertEquals(1, pm.getPlacementRules().size());
+    // Verifying if placement rule set is same as the one we configured
+    Assert.assertEquals(ugRule.getName(),
+        pm.getPlacementRules().get(0).getName());
+  }
+
+  @After
+  public void tearDown() {
+    if (null != mockRM) {
+      mockRM.stop();
+    }
+  }
+
 }