ソースを参照

YARN-3660. [GPG] Federation Global Policy Generator (service hook only). (Contributed by Botong Huang via curino)

Carlo Curino 7 年 前
コミット
af0c804b9f

+ 6 - 0
hadoop-project/pom.xml

@@ -451,6 +451,12 @@
         <version>${hadoop.version}</version>
       </dependency>
 
+      <dependency>
+        <groupId>org.apache.hadoop</groupId>
+        <artifactId>hadoop-yarn-server-globalpolicygenerator</artifactId>
+        <version>${project.version}</version>
+      </dependency>
+
       <dependency>
         <groupId>org.apache.hadoop</groupId>
         <artifactId>hadoop-yarn-services-core</artifactId>

+ 5 - 0
hadoop-yarn-project/hadoop-yarn/bin/yarn

@@ -39,6 +39,7 @@ function hadoop_usage
   hadoop_add_subcommand "container" client "prints container(s) report"
   hadoop_add_subcommand "daemonlog" admin "get/set the log level for each daemon"
   hadoop_add_subcommand "envvars" client "display computed Hadoop environment variables"
+  hadoop_add_subcommand "globalpolicygenerator" daemon "run the Global Policy Generator"
   hadoop_add_subcommand "jar <jar>" client "run a jar file"
   hadoop_add_subcommand "logs" client "dump container logs"
   hadoop_add_subcommand "node" admin "prints node report(s)"
@@ -104,6 +105,10 @@ ${HADOOP_COMMON_HOME}/${HADOOP_COMMON_LIB_JARS_DIR}"
       echo "HADOOP_TOOLS_LIB_JARS_DIR='${HADOOP_TOOLS_LIB_JARS_DIR}'"
       exit 0
     ;;
+    globalpolicygenerator)
+      HADOOP_SUBCMD_SUPPORTDAEMONIZATION="true"
+      HADOOP_CLASSNAME='org.apache.hadoop.yarn.server.globalpolicygenerator.GlobalPolicyGenerator'
+    ;;
     jar)
       HADOOP_CLASSNAME=org.apache.hadoop.util.RunJar
     ;;

+ 33 - 22
hadoop-yarn-project/hadoop-yarn/bin/yarn.cmd

@@ -134,6 +134,10 @@ if "%1" == "--loglevel" (
     set CLASSPATH=%CLASSPATH%;%HADOOP_YARN_HOME%\yarn-server\yarn-server-router\target\classes
   )
 
+  if exist %HADOOP_YARN_HOME%\yarn-server\yarn-server-globalpolicygenerator\target\classes (
+    set CLASSPATH=%CLASSPATH%;%HADOOP_YARN_HOME%\yarn-server\yarn-server-globalpolicygenerator\target\classes
+  )
+
   if exist %HADOOP_YARN_HOME%\build\test\classes (
     set CLASSPATH=%CLASSPATH%;%HADOOP_YARN_HOME%\build\test\classes
   )
@@ -155,7 +159,7 @@ if "%1" == "--loglevel" (
 
   set yarncommands=resourcemanager nodemanager proxyserver rmadmin version jar ^
      application applicationattempt container node queue logs daemonlog historyserver ^
-     timelineserver timelinereader router classpath
+     timelineserver timelinereader router globalpolicygenerator classpath
   for %%i in ( %yarncommands% ) do (
     if %yarn-command% == %%i set yarncommand=true
   )
@@ -259,7 +263,13 @@ goto :eof
 :router
   set CLASSPATH=%CLASSPATH%;%YARN_CONF_DIR%\router-config\log4j.properties
   set CLASS=org.apache.hadoop.yarn.server.router.Router
-  set YARN_OPTS=%YARN_OPTS% %HADOOP_ROUTER_OPTS%
+  set YARN_OPTS=%YARN_OPTS% %YARN_ROUTER_OPTS%
+  goto :eof
+
+:globalpolicygenerator
+  set CLASSPATH=%CLASSPATH%;%YARN_CONF_DIR%\globalpolicygenerator-config\log4j.properties
+  set CLASS=org.apache.hadoop.yarn.server.globalpolicygenerator.GlobalPolicyGenerator
+  set YARN_OPTS=%YARN_OPTS% %YARN_GLOBALPOLICYGENERATOR_OPTS%
   goto :eof
 
 :nodemanager
@@ -336,27 +346,28 @@ goto :eof
 :print_usage
   @echo Usage: yarn [--config confdir] [--loglevel loglevel] COMMAND
   @echo        where COMMAND is one of:
-  @echo   resourcemanager      run the ResourceManager
-  @echo   nodemanager          run a nodemanager on each slave
-  @echo   router               run the Router daemon
-  @echo   timelineserver       run the timeline server
-  @echo   timelinereader       run the timeline reader server
-  @echo   rmadmin              admin tools
-  @echo   version              print the version
-  @echo   jar ^<jar^>          run a jar file
-  @echo   application          prints application(s) report/kill application
-  @echo   applicationattempt   prints applicationattempt(s) report
-  @echo   cluster              prints cluster information
-  @echo   container            prints container(s) report
-  @echo   node                 prints node report(s)
-  @echo   queue                prints queue information
-  @echo   logs                 dump container logs
-  @echo   schedulerconf        updates scheduler configuration
-  @echo   classpath            prints the class path needed to get the
-  @echo                        Hadoop jar and the required libraries
-  @echo   daemonlog            get/set the log level for each daemon
+  @echo   resourcemanager        run the ResourceManager
+  @echo   nodemanager            run a nodemanager on each slave
+  @echo   router                 run the Router daemon
+  @echo   globalpolicygenerator  run the Global Policy Generator
+  @echo   timelineserver         run the timeline server
+  @echo   timelinereader         run the timeline reader server
+  @echo   rmadmin                admin tools
+  @echo   version                print the version
+  @echo   jar ^<jar^>            run a jar file
+  @echo   application            prints application(s) report/kill application
+  @echo   applicationattempt     prints applicationattempt(s) report
+  @echo   cluster                prints cluster information
+  @echo   container              prints container(s) report
+  @echo   node                   prints node report(s)
+  @echo   queue                  prints queue information
+  @echo   logs                   dump container logs
+  @echo   schedulerconf          updates scheduler configuration
+  @echo   classpath              prints the class path needed to get the
+  @echo                          Hadoop jar and the required libraries
+  @echo   daemonlog              get/set the log level for each daemon
   @echo   or
-  @echo   CLASSNAME            run the class named CLASSNAME
+  @echo   CLASSNAME              run the class named CLASSNAME
   @echo Most commands print help when invoked w/o parameters.
 
 endlocal

+ 12 - 0
hadoop-yarn-project/hadoop-yarn/conf/yarn-env.sh

@@ -149,6 +149,18 @@
 #
 #export YARN_ROUTER_OPTS=
 
+###
+# Global Policy Generator specific parameters
+###
+
+# Specify the JVM options to be used when starting the GPG.
+# These options will be appended to the options specified as HADOOP_OPTS
+# and therefore may override any similar flags set in HADOOP_OPTS
+#
+# See ResourceManager for some examples
+#
+#export YARN_GLOBALPOLICYGENERATOR_OPTS=
+
 ###
 # Registry DNS specific parameters
 # This is deprecated and should be done in hadoop-env.sh

+ 98 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-globalpolicygenerator/pom.xml

@@ -0,0 +1,98 @@
+<?xml version="1.0"?>
+<!--
+  Licensed 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. See accompanying LICENSE file.
+-->
+<project xmlns="http://maven.apache.org/POM/4.0.0"
+  xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
+  xsi:schemaLocation="http://maven.apache.org/POM/4.0.0
+                      http://maven.apache.org/xsd/maven-4.0.0.xsd">
+  <parent>
+    <artifactId>hadoop-yarn-server</artifactId>
+    <groupId>org.apache.hadoop</groupId>
+    <version>3.1.0-SNAPSHOT</version>
+  </parent>
+  <modelVersion>4.0.0</modelVersion>
+  <groupId>org.apache.hadoop</groupId>
+  <artifactId>hadoop-yarn-server-globalpolicygenerator</artifactId>
+  <version>3.1.0-SNAPSHOT</version>
+  <name>hadoop-yarn-server-globalpolicygenerator</name>
+
+  <properties>
+    <!-- Needed for generating FindBugs warnings using parent pom -->
+    <yarn.basedir>${project.parent.parent.basedir}</yarn.basedir>
+  </properties>
+
+  <dependencies>
+
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-yarn-server-common</artifactId>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-common</artifactId>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-yarn-api</artifactId>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-yarn-common</artifactId>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-common</artifactId>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-yarn-server-resourcemanager</artifactId>
+    </dependency>
+
+    <dependency>
+      <groupId>junit</groupId>
+      <artifactId>junit</artifactId>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-yarn-server-common</artifactId>
+      <type>test-jar</type>
+      <scope>test</scope>
+    </dependency>
+
+    <dependency>
+      <groupId>org.hsqldb</groupId>
+      <artifactId>hsqldb</artifactId>
+      <scope>test</scope>
+    </dependency>
+
+  </dependencies>
+
+  <build>
+    <plugins>
+      <plugin>
+        <groupId>org.apache.rat</groupId>
+        <artifactId>apache-rat-plugin</artifactId>
+      </plugin>
+    </plugins>
+  </build>
+</project>

+ 31 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-globalpolicygenerator/src/main/java/org/apache/hadoop/yarn/server/globalpolicygenerator/GPGContext.java

@@ -0,0 +1,31 @@
+/**
+ *  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.globalpolicygenerator;
+
+import org.apache.hadoop.yarn.server.federation.utils.FederationStateStoreFacade;
+
+/**
+ * Context for Global Policy Generator.
+ */
+public interface GPGContext {
+
+  FederationStateStoreFacade getStateStoreFacade();
+
+  void setStateStoreFacade(FederationStateStoreFacade facade);
+}

+ 41 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-globalpolicygenerator/src/main/java/org/apache/hadoop/yarn/server/globalpolicygenerator/GPGContextImpl.java

@@ -0,0 +1,41 @@
+/**
+ *  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.globalpolicygenerator;
+
+import org.apache.hadoop.yarn.server.federation.utils.FederationStateStoreFacade;
+
+/**
+ * Context implementation for Global Policy Generator.
+ */
+public class GPGContextImpl implements GPGContext {
+
+  private FederationStateStoreFacade facade;
+
+  @Override
+  public FederationStateStoreFacade getStateStoreFacade() {
+    return facade;
+  }
+
+  @Override
+  public void setStateStoreFacade(
+      FederationStateStoreFacade federationStateStoreFacade) {
+    this.facade = federationStateStoreFacade;
+  }
+
+}

+ 136 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-globalpolicygenerator/src/main/java/org/apache/hadoop/yarn/server/globalpolicygenerator/GlobalPolicyGenerator.java

@@ -0,0 +1,136 @@
+/**
+ * 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.globalpolicygenerator;
+
+import java.util.concurrent.atomic.AtomicBoolean;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.metrics2.lib.DefaultMetricsSystem;
+import org.apache.hadoop.service.CompositeService;
+import org.apache.hadoop.util.ShutdownHookManager;
+import org.apache.hadoop.util.StringUtils;
+import org.apache.hadoop.yarn.YarnUncaughtExceptionHandler;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
+import org.apache.hadoop.yarn.server.federation.utils.FederationStateStoreFacade;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * Global Policy Generator (GPG) is a Yarn Federation component. By tuning the
+ * Federation policies in Federation State Store, GPG overlooks the entire
+ * federated cluster and ensures that the system is tuned and balanced all the
+ * time.
+ *
+ * The GPG operates continuously but out-of-band from all cluster operations,
+ * that allows to enforce global invariants, affect load balancing, trigger
+ * draining of sub-clusters that will undergo maintenance, etc.
+ */
+public class GlobalPolicyGenerator extends CompositeService {
+
+  public static final Logger LOG =
+      LoggerFactory.getLogger(GlobalPolicyGenerator.class);
+
+  // YARN Variables
+  private static CompositeServiceShutdownHook gpgShutdownHook;
+  public static final int SHUTDOWN_HOOK_PRIORITY = 30;
+  private AtomicBoolean isStopping = new AtomicBoolean(false);
+  private static final String METRICS_NAME = "Global Policy Generator";
+
+  // Federation Variables
+  private GPGContext gpgContext;
+
+  public GlobalPolicyGenerator() {
+    super(GlobalPolicyGenerator.class.getName());
+    this.gpgContext = new GPGContextImpl();
+  }
+
+  protected void initAndStart(Configuration conf, boolean hasToReboot) {
+    try {
+      // Remove the old hook if we are rebooting.
+      if (hasToReboot && null != gpgShutdownHook) {
+        ShutdownHookManager.get().removeShutdownHook(gpgShutdownHook);
+      }
+
+      gpgShutdownHook = new CompositeServiceShutdownHook(this);
+      ShutdownHookManager.get().addShutdownHook(gpgShutdownHook,
+          SHUTDOWN_HOOK_PRIORITY);
+
+      this.init(conf);
+      this.start();
+    } catch (Throwable t) {
+      LOG.error("Error starting globalpolicygenerator", t);
+      System.exit(-1);
+    }
+  }
+
+  @Override
+  protected void serviceInit(Configuration conf) throws Exception {
+    // Set up the context
+    this.gpgContext
+        .setStateStoreFacade(FederationStateStoreFacade.getInstance());
+
+    DefaultMetricsSystem.initialize(METRICS_NAME);
+
+    // super.serviceInit after all services are added
+    super.serviceInit(conf);
+  }
+
+  @Override
+  protected void serviceStart() throws Exception {
+    super.serviceStart();
+  }
+
+  @Override
+  protected void serviceStop() throws Exception {
+    if (this.isStopping.getAndSet(true)) {
+      return;
+    }
+    DefaultMetricsSystem.shutdown();
+    super.serviceStop();
+  }
+
+  public String getName() {
+    return "FederationGlobalPolicyGenerator";
+  }
+
+  public GPGContext getGPGContext() {
+    return this.gpgContext;
+  }
+
+  @SuppressWarnings("resource")
+  public static void startGPG(String[] argv, Configuration conf) {
+    boolean federationEnabled =
+        conf.getBoolean(YarnConfiguration.FEDERATION_ENABLED,
+            YarnConfiguration.DEFAULT_FEDERATION_ENABLED);
+    if (federationEnabled) {
+      Thread.setDefaultUncaughtExceptionHandler(
+          new YarnUncaughtExceptionHandler());
+      StringUtils.startupShutdownMessage(GlobalPolicyGenerator.class, argv,
+          LOG);
+      GlobalPolicyGenerator globalPolicyGenerator = new GlobalPolicyGenerator();
+      globalPolicyGenerator.initAndStart(conf, false);
+    } else {
+      LOG.warn("Federation is not enabled. The gpg cannot start.");
+    }
+  }
+
+  public static void main(String[] argv) {
+    startGPG(argv, new YarnConfiguration());
+  }
+}

+ 19 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-globalpolicygenerator/src/main/java/org/apache/hadoop/yarn/server/globalpolicygenerator/package-info.java

@@ -0,0 +1,19 @@
+/**
+ *  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.globalpolicygenerator;

+ 38 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-globalpolicygenerator/src/test/java/org/apache/hadoop/yarn/server/globalpolicygenerator/TestGlobalPolicyGenerator.java

@@ -0,0 +1,38 @@
+/**
+ *  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.globalpolicygenerator;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
+import org.junit.Test;
+
+/**
+ * Unit test for GlobalPolicyGenerator.
+ */
+public class TestGlobalPolicyGenerator {
+
+  @Test(timeout = 1000)
+  public void testNonFederation() {
+    Configuration conf = new YarnConfiguration();
+    conf.setBoolean(YarnConfiguration.FEDERATION_ENABLED, false);
+
+    // If GPG starts running, this call will not return
+    GlobalPolicyGenerator.startGPG(new String[0], conf);
+  }
+}

+ 1 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/pom.xml

@@ -46,5 +46,6 @@
     <module>hadoop-yarn-server-timelineservice-hbase</module>
     <module>hadoop-yarn-server-timelineservice-hbase-tests</module>
     <module>hadoop-yarn-server-router</module>
+    <module>hadoop-yarn-server-globalpolicygenerator</module>
   </modules>
 </project>

+ 4 - 0
hadoop-yarn-project/pom.xml

@@ -78,6 +78,10 @@
       <groupId>org.apache.hadoop</groupId>
       <artifactId>hadoop-yarn-server-router</artifactId>
     </dependency>
+    <dependency>
+      <groupId>org.apache.hadoop</groupId>
+      <artifactId>hadoop-yarn-server-globalpolicygenerator</artifactId>
+    </dependency>
     <dependency>
       <groupId>org.apache.hadoop</groupId>
       <artifactId>hadoop-yarn-services-core</artifactId>