Browse Source

YARN-1374. Changed ResourceManager to start the preemption policy monitors as active services. Contributed by Karthik Kambatla.
svn merge --ignore-ancestry -c 1539089 ../../trunk/


git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-2@1539090 13f79535-47bb-0310-9956-ffa450edef68

Vinod Kumar Vavilapalli 11 years ago
parent
commit
c8d2faab1b

+ 3 - 0
hadoop-yarn-project/CHANGES.txt

@@ -91,6 +91,9 @@ Release 2.3.0 - UNRELEASED
     YARN-1305. RMHAProtocolService#serviceInit should handle HAUtil's
     IllegalArgumentException (Tsuyoshi Ozawa via bikas)
 
+    YARN-1374. Changed ResourceManager to start the preemption policy monitors
+    as active services. (Karthik Kambatla via vinodkv)
+
 Release 2.2.1 - UNRELEASED
 
   INCOMPATIBLE CHANGES

+ 30 - 31
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/ResourceManager.java

@@ -501,6 +501,36 @@ public class ResourceManager extends CompositeService implements Recoverable {
 
       super.serviceStop();
     }
+
+    protected void createPolicyMonitors() {
+      if (scheduler instanceof PreemptableResourceScheduler
+          && conf.getBoolean(YarnConfiguration.RM_SCHEDULER_ENABLE_MONITORS,
+          YarnConfiguration.DEFAULT_RM_SCHEDULER_ENABLE_MONITORS)) {
+        LOG.info("Loading policy monitors");
+        List<SchedulingEditPolicy> policies = conf.getInstances(
+            YarnConfiguration.RM_SCHEDULER_MONITOR_POLICIES,
+            SchedulingEditPolicy.class);
+        if (policies.size() > 0) {
+          rmDispatcher.register(ContainerPreemptEventType.class,
+              new RMContainerPreemptEventDispatcher(
+                  (PreemptableResourceScheduler) scheduler));
+          for (SchedulingEditPolicy policy : policies) {
+            LOG.info("LOADING SchedulingEditPolicy:" + policy.getPolicyName());
+            policy.init(conf, rmContext.getDispatcher().getEventHandler(),
+                (PreemptableResourceScheduler) scheduler);
+            // periodically check whether we need to take action to guarantee
+            // constraints
+            SchedulingMonitor mon = new SchedulingMonitor(policy);
+            addService(mon);
+          }
+        } else {
+          LOG.warn("Policy monitors configured (" +
+              YarnConfiguration.RM_SCHEDULER_ENABLE_MONITORS +
+              ") but none specified (" +
+              YarnConfiguration.RM_SCHEDULER_MONITOR_POLICIES + ")");
+        }
+      }
+    }
   }
 
   @Private
@@ -829,37 +859,6 @@ public class ResourceManager extends CompositeService implements Recoverable {
     return new ApplicationMasterService(this.rmContext, scheduler);
   }
 
-  protected void createPolicyMonitors() {
-    if (scheduler instanceof PreemptableResourceScheduler
-        && conf.getBoolean(YarnConfiguration.RM_SCHEDULER_ENABLE_MONITORS,
-          YarnConfiguration.DEFAULT_RM_SCHEDULER_ENABLE_MONITORS)) {
-      LOG.info("Loading policy monitors");
-      List<SchedulingEditPolicy> policies = conf.getInstances(
-              YarnConfiguration.RM_SCHEDULER_MONITOR_POLICIES,
-              SchedulingEditPolicy.class);
-      if (policies.size() > 0) {
-        this.rmDispatcher.register(ContainerPreemptEventType.class,
-          new RMContainerPreemptEventDispatcher(
-            (PreemptableResourceScheduler) scheduler));
-        for (SchedulingEditPolicy policy : policies) {
-          LOG.info("LOADING SchedulingEditPolicy:" + policy.getPolicyName());
-          policy.init(conf, this.rmContext.getDispatcher().getEventHandler(),
-              (PreemptableResourceScheduler) scheduler);
-          // periodically check whether we need to take action to guarantee
-          // constraints
-          SchedulingMonitor mon = new SchedulingMonitor(policy);
-          addService(mon);
-
-        }
-      } else {
-        LOG.warn("Policy monitors configured (" +
-            YarnConfiguration.RM_SCHEDULER_ENABLE_MONITORS +
-            ") but none specified (" +
-            YarnConfiguration.RM_SCHEDULER_MONITOR_POLICIES + ")");
-      }
-    }
-  }
-
   protected AdminService createAdminService(
       ClientRMService clientRMService, 
       ApplicationMasterService applicationMasterService,

+ 46 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/test/java/org/apache/hadoop/yarn/server/resourcemanager/monitor/TestSchedulingMonitor.java

@@ -0,0 +1,46 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.yarn.server.resourcemanager.monitor;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
+import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
+import org.apache.hadoop.yarn.server.resourcemanager.monitor.capacity.ProportionalCapacityPreemptionPolicy;
+import org.junit.Test;
+
+import static org.junit.Assert.fail;
+
+public class TestSchedulingMonitor {
+
+  @Test(timeout = 10000)
+  public void testRMStarts() {
+    Configuration conf = new YarnConfiguration();
+    conf.setBoolean(YarnConfiguration.RM_SCHEDULER_ENABLE_MONITORS, true);
+    conf.set(YarnConfiguration.RM_SCHEDULER_MONITOR_POLICIES,
+        ProportionalCapacityPreemptionPolicy.class.getCanonicalName());
+
+    ResourceManager rm = new ResourceManager();
+    try {
+      rm.init(conf);
+    } catch (Exception e) {
+      fail("ResourceManager does not start when " +
+          YarnConfiguration.RM_SCHEDULER_ENABLE_MONITORS + " is set to true");
+    }
+  }
+}