|
@@ -32,6 +32,14 @@ import org.apache.hadoop.yarn.exceptions.YarnException;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.placement.UserGroupMappingPlacementRule.QueueMapping.MappingType;
|
|
import org.apache.hadoop.yarn.server.resourcemanager.placement.UserGroupMappingPlacementRule.QueueMapping.MappingType;
|
|
|
|
|
|
import com.google.common.annotations.VisibleForTesting;
|
|
import com.google.common.annotations.VisibleForTesting;
|
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler;
|
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CSQueue;
|
|
|
|
+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.server.resourcemanager.scheduler.capacity.CapacitySchedulerContext;
|
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.CapacitySchedulerQueueManager;
|
|
|
|
+import org.apache.hadoop.yarn.server.resourcemanager.scheduler.capacity.LeafQueue;
|
|
|
|
+
|
|
|
|
|
|
public class UserGroupMappingPlacementRule extends PlacementRule {
|
|
public class UserGroupMappingPlacementRule extends PlacementRule {
|
|
private static final Log LOG = LogFactory
|
|
private static final Log LOG = LogFactory
|
|
@@ -95,6 +103,10 @@ public class UserGroupMappingPlacementRule extends PlacementRule {
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ public UserGroupMappingPlacementRule(){
|
|
|
|
+ this(false, null, null);
|
|
|
|
+ }
|
|
|
|
+
|
|
public UserGroupMappingPlacementRule(boolean overrideWithQueueMappings,
|
|
public UserGroupMappingPlacementRule(boolean overrideWithQueueMappings,
|
|
List<QueueMapping> newMappings, Groups groups) {
|
|
List<QueueMapping> newMappings, Groups groups) {
|
|
this.mappings = newMappings;
|
|
this.mappings = newMappings;
|
|
@@ -156,7 +168,55 @@ public class UserGroupMappingPlacementRule extends PlacementRule {
|
|
|
|
|
|
return queueName;
|
|
return queueName;
|
|
}
|
|
}
|
|
-
|
|
|
|
|
|
+
|
|
|
|
+ @VisibleForTesting
|
|
|
|
+ @Override
|
|
|
|
+ public boolean initialize(ResourceScheduler scheduler)
|
|
|
|
+ throws IOException {
|
|
|
|
+ if (!(scheduler instanceof CapacityScheduler)) {
|
|
|
|
+ throw new IOException(
|
|
|
|
+ "UserGroupMappingPlacementRule can be configured only for "
|
|
|
|
+ + "CapacityScheduler");
|
|
|
|
+ }
|
|
|
|
+ CapacitySchedulerContext schedulerContext =
|
|
|
|
+ (CapacitySchedulerContext) scheduler;
|
|
|
|
+ CapacitySchedulerConfiguration conf = schedulerContext.getConfiguration();
|
|
|
|
+ boolean overrideWithQueueMappings = conf.getOverrideWithQueueMappings();
|
|
|
|
+ LOG.info(
|
|
|
|
+ "Initialized queue mappings, override: " + overrideWithQueueMappings);
|
|
|
|
+
|
|
|
|
+ // Get new user/group mappings
|
|
|
|
+ List<QueueMapping> newMappings = conf.getQueueMappings();
|
|
|
|
+
|
|
|
|
+ CapacitySchedulerQueueManager queueManager =
|
|
|
|
+ schedulerContext.getCapacitySchedulerQueueManager();
|
|
|
|
+
|
|
|
|
+ // check if mappings refer to valid queues
|
|
|
|
+ for (QueueMapping mapping : newMappings) {
|
|
|
|
+ String mappingQueue = mapping.getQueue();
|
|
|
|
+ if (!mappingQueue.equals(
|
|
|
|
+ UserGroupMappingPlacementRule.CURRENT_USER_MAPPING) && !mappingQueue
|
|
|
|
+ .equals(UserGroupMappingPlacementRule.PRIMARY_GROUP_MAPPING)) {
|
|
|
|
+ CSQueue queue = queueManager.getQueue(mappingQueue);
|
|
|
|
+ if (queue == null || !(queue instanceof LeafQueue)) {
|
|
|
|
+ throw new IOException(
|
|
|
|
+ "mapping contains invalid or non-leaf queue " + mappingQueue);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ // initialize groups if mappings are present
|
|
|
|
+ if (newMappings.size() > 0) {
|
|
|
|
+ Groups groups = new Groups(conf);
|
|
|
|
+ this.mappings = newMappings;
|
|
|
|
+ this.groups = groups;
|
|
|
|
+ this.overrideWithQueueMappings = overrideWithQueueMappings;
|
|
|
|
+ return true;
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ return false;
|
|
|
|
+ }
|
|
|
|
+
|
|
@VisibleForTesting
|
|
@VisibleForTesting
|
|
public List<QueueMapping> getQueueMappings() {
|
|
public List<QueueMapping> getQueueMappings() {
|
|
return mappings;
|
|
return mappings;
|