浏览代码

YARN-7091. Rename application to service in yarn-native-services. Contributed by Jian He

Billie Rinaldi 7 年之前
父节点
当前提交
40ab068eab
共有 100 个文件被更改,包括 832 次插入805 次删除
  1. 4 4
      hadoop-assemblies/src/main/resources/assemblies/hadoop-yarn-dist.xml
  2. 1 1
      hadoop-assemblies/src/main/resources/assemblies/hadoop-yarn-services-dist.xml
  3. 1 1
      hadoop-project/pom.xml
  4. 7 7
      hadoop-yarn-project/hadoop-yarn/bin/yarn
  5. 1 1
      hadoop-yarn-project/hadoop-yarn/conf/yarn-env.sh
  6. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/pom.xml
  7. 81 82
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/java/org/apache/hadoop/yarn/service/webapp/ApiServer.java
  8. 159 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/java/org/apache/hadoop/yarn/service/webapp/ApiServerWebApp.java
  9. 0 123
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/java/org/apache/hadoop/yarn/service/webapp/ApplicationApiWebApp.java
  10. 93 92
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/resources/definition/YARN-Simplified-V1-API-Layer-For-Services.yaml
  11. 0 28
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/scripts/run_rest_service.sh
  12. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/dev-support/findbugs-exclude.xml
  13. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/conf/yarnservice-log4j.properties
  14. 3 4
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/pom.xml
  15. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ClientAMProtocol.java
  16. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ClientAMService.java
  17. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ContainerFailureTracker.java
  18. 2 2
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ServiceContext.java
  19. 5 6
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ServiceMaster.java
  20. 1 4
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ServiceMetrics.java
  21. 15 16
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ServiceScheduler.java
  22. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/ServiceApiConstants.java
  23. 5 5
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Artifact.java
  24. 3 3
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/BaseResource.java
  25. 26 26
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Component.java
  26. 4 4
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ConfigFile.java
  27. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ConfigFormat.java
  28. 8 8
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Configuration.java
  29. 6 6
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Container.java
  30. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ContainerState.java
  31. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Error.java
  32. 7 7
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/PlacementPolicy.java
  33. 5 5
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ReadinessCheck.java
  34. 5 5
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Resource.java
  35. 66 66
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Service.java
  36. 3 3
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ServiceState.java
  37. 20 20
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ServiceStatus.java
  38. 13 5
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/ClientAMProxy.java
  39. 11 3
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/ServiceCLI.java
  40. 199 179
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/ServiceClient.java
  41. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractActionArgs.java
  42. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractArgsDelegate.java
  43. 3 3
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractClusterBuildingActionArgs.java
  44. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionBuildArgs.java
  45. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionClientArgs.java
  46. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionCreateArgs.java
  47. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionDependencyArgs.java
  48. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionDestroyArgs.java
  49. 3 3
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionExistsArgs.java
  50. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionFlexArgs.java
  51. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionFreezeArgs.java
  52. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionHelpArgs.java
  53. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionKDiagArgs.java
  54. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionKeytabArgs.java
  55. 2 2
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionListArgs.java
  56. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionRegistryArgs.java
  57. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionResolveArgs.java
  58. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionResourceArgs.java
  59. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionStatusArgs.java
  60. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionThawArgs.java
  61. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionTokensArgs.java
  62. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionUpdateArgs.java
  63. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ArgOps.java
  64. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/Arguments.java
  65. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ClientArgs.java
  66. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/CommonArgs.java
  67. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ComponentArgsDelegate.java
  68. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/DontSplitArguments.java
  69. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/LaunchArgsAccessor.java
  70. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/LaunchArgsDelegate.java
  71. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/OptionArgsDelegate.java
  72. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/PathArgumentConverter.java
  73. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderAMArgs.java
  74. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderAMCreateAction.java
  75. 2 2
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderActions.java
  76. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/WaitArgsDelegate.java
  77. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/WaitTimeAccessor.java
  78. 14 13
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/Component.java
  79. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentEvent.java
  80. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentEventType.java
  81. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentState.java
  82. 10 10
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstance.java
  83. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstanceEvent.java
  84. 2 3
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstanceEventType.java
  85. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstanceId.java
  86. 2 2
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstanceState.java
  87. 6 6
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/conf/RestApiConstants.java
  88. 4 4
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/conf/SliderExitCodes.java
  89. 4 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/conf/YarnServiceConf.java
  90. 1 1
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/conf/YarnServiceConstants.java
  91. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/AbstractLauncher.java
  92. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/ClasspathConstructor.java
  93. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/CommandLineBuilder.java
  94. 8 8
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/ContainerLaunchService.java
  95. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/CredentialUtils.java
  96. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/JavaCommandLineBuilder.java
  97. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadClusterStateException.java
  98. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadCommandArgumentsException.java
  99. 0 0
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadConfigException.java
  100. 5 20
      hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/ErrorStrings.java

+ 4 - 4
hadoop-assemblies/src/main/resources/assemblies/hadoop-yarn-dist.xml

@@ -87,19 +87,19 @@
       </includes>
     </fileSet>
     <fileSet>
-      <directory>hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/target</directory>
+      <directory>hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/target</directory>
       <outputDirectory>/share/hadoop/${hadoop.component}/sources</outputDirectory>
       <includes>
         <include>*-sources.jar</include>
       </includes>
     </fileSet>
     <fileSet>
-      <directory>hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/conf</directory>
+      <directory>hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/conf</directory>
       <outputDirectory>etc/hadoop</outputDirectory>
     </fileSet>
     <fileSet>
-      <directory>hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/target/hadoop-yarn-slider-core-${project.version}</directory>
-      <outputDirectory>/share/hadoop/${hadoop.component}/lib/slider</outputDirectory>
+      <directory>hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/target/hadoop-yarn-services-core-${project.version}</directory>
+      <outputDirectory>/share/hadoop/${hadoop.component}/lib/services</outputDirectory>
     </fileSet>
     <fileSet>
       <directory>hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/target</directory>

+ 1 - 1
hadoop-assemblies/src/main/resources/assemblies/hadoop-yarn-services-dist.xml

@@ -17,7 +17,7 @@
 <assembly xmlns="http://maven.apache.org/plugins/maven-assembly-plugin/assembly/1.1.0"
           xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance"
           xsi:schemaLocation="http://maven.apache.org/plugins/maven-assembly-plugin/assembly/1.1.0 http://maven.apache.org/xsd/assembly-1.1.0.xsd">
-  <id>hadoop-yarn-slider-dist</id>
+  <id>hadoop-yarn-services-dist</id>
   <formats>
     <format>dir</format>
   </formats>

+ 1 - 1
hadoop-project/pom.xml

@@ -428,7 +428,7 @@
 
       <dependency>
         <groupId>org.apache.hadoop</groupId>
-        <artifactId>hadoop-yarn-slider-core</artifactId>
+        <artifactId>hadoop-yarn-services-core</artifactId>
         <version>${project.version}</version>
       </dependency>
 

+ 7 - 7
hadoop-yarn-project/hadoop-yarn/bin/yarn

@@ -49,7 +49,7 @@ function hadoop_usage
   hadoop_add_subcommand "router" daemon "run the Router daemon"
   hadoop_add_subcommand "schedulerconf" client "Updates scheduler configuration"
   hadoop_add_subcommand "scmadmin" admin "SharedCacheManager admin tools"
-  hadoop_add_subcommand "servicesapi" "run yarn-native-service rest server"
+  hadoop_add_subcommand "apiserver" "run yarn-native-service rest server"
   hadoop_add_subcommand "sharedcachemanager" daemon "run the SharedCacheManager daemon"
   hadoop_add_subcommand "service" "run a service"
   hadoop_add_subcommand "timelinereader" client "run the timeline reader server"
@@ -151,14 +151,14 @@ function yarncmd_case
     scmadmin)
       HADOOP_CLASSNAME='org.apache.hadoop.yarn.client.SCMAdmin'
     ;;
-    servicesapi)
+    apiserver)
       HADOOP_SUBCMD_SUPPORTDAEMONIZATION="true"
-      hadoop_add_classpath "${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/slider"'/*'
+      hadoop_add_classpath "${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/services"'/*'
       hadoop_add_classpath "${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/services-api"'/*'
-      HADOOP_CLASSNAME='org.apache.hadoop.yarn.service.webapp.ApplicationApiWebApp'
+      HADOOP_CLASSNAME='org.apache.hadoop.yarn.service.webapp.ApiServerWebApp'
       local sld="${HADOOP_YARN_HOME}/${YARN_DIR},\
 ${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR},\
-${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/slider,\
+${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/services,\
 ${HADOOP_HDFS_HOME}/${HDFS_DIR},\
 ${HADOOP_HDFS_HOME}/${HDFS_LIB_JARS_DIR},\
 ${HADOOP_COMMON_HOME}/${HADOOP_COMMON_DIR},\
@@ -171,11 +171,11 @@ ${HADOOP_COMMON_HOME}/${HADOOP_COMMON_LIB_JARS_DIR}"
       HADOOP_CLASSNAME='org.apache.hadoop.yarn.server.sharedcachemanager.SharedCacheManager'
     ;;
     service)
-      hadoop_add_classpath "${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/slider"'/*'
+      hadoop_add_classpath "${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/services"'/*'
       HADOOP_CLASSNAME='org.apache.hadoop.yarn.service.client.ServiceCLI'
       local sld="${HADOOP_YARN_HOME}/${YARN_DIR},\
 ${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR},\
-${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/slider,\
+${HADOOP_YARN_HOME}/${YARN_LIB_JARS_DIR}/services,\
 ${HADOOP_HDFS_HOME}/${HDFS_DIR},\
 ${HADOOP_HDFS_HOME}/${HDFS_LIB_JARS_DIR},\
 ${HADOOP_COMMON_HOME}/${HADOOP_COMMON_DIR},\

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/conf/yarn-env.sh

@@ -159,4 +159,4 @@
 #
 # See ResourceManager for some examples
 #
-#export YARN_SERVICESAPI_OPTS="-verbose:gc -XX:+PrintGCDetails -XX:+PrintGCTimeStamps -XX:+PrintGCDateStamps -Xloggc:${HADOOP_LOG_DIR}/gc-servicesapi.log-$(date +'%Y%m%d%H%M')"
+#export YARN_APISERVER_OPTS="-verbose:gc -XX:+PrintGCDetails -XX:+PrintGCTimeStamps -XX:+PrintGCDateStamps -Xloggc:${HADOOP_LOG_DIR}/gc-apiserver.log-$(date +'%Y%m%d%H%M')"

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/pom.xml

@@ -91,7 +91,7 @@
   <dependencies>
     <dependency>
       <groupId>org.apache.hadoop</groupId>
-      <artifactId>hadoop-yarn-slider-core</artifactId>
+      <artifactId>hadoop-yarn-services-core</artifactId>
       <version>${project.version}</version>
     </dependency>
     <dependency>

+ 81 - 82
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/java/org/apache/hadoop/yarn/service/webapp/ApplicationApiService.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/java/org/apache/hadoop/yarn/service/webapp/ApiServer.java

@@ -24,10 +24,10 @@ import org.apache.hadoop.yarn.api.records.ApplicationId;
 import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.exceptions.ApplicationNotFoundException;
 import org.apache.hadoop.yarn.exceptions.YarnException;
+import org.apache.hadoop.yarn.service.api.records.Service;
+import org.apache.hadoop.yarn.service.api.records.ServiceState;
+import org.apache.hadoop.yarn.service.api.records.ServiceStatus;
 import org.apache.hadoop.yarn.service.client.ServiceClient;
-import org.apache.hadoop.yarn.service.api.records.Application;
-import org.apache.hadoop.yarn.service.api.records.ApplicationState;
-import org.apache.hadoop.yarn.service.api.records.ApplicationStatus;
 import org.apache.hadoop.yarn.service.api.records.Component;
 import org.apache.hadoop.yarn.service.utils.SliderUtils;
 import org.apache.hadoop.yarn.service.utils.ServiceApiUtil;
@@ -51,11 +51,14 @@ import java.util.Map;
 
 import static org.apache.hadoop.yarn.service.conf.RestApiConstants.*;
 
+/**
+ * The rest API endpoints for users to manage services on YARN.
+ */
 @Singleton
 @Path(CONTEXT_ROOT)
-public class ApplicationApiService {
+public class ApiServer {
   private static final Logger LOG =
-      LoggerFactory.getLogger(ApplicationApiService.class);
+      LoggerFactory.getLogger(ApiServer.class);
   private static Configuration YARN_CONFIG = new YarnConfiguration();
   private static ServiceClient SERVICE_CLIENT;
 
@@ -81,92 +84,88 @@ public class ApplicationApiService {
   }
 
   @POST
-  @Path(APP_ROOT_PATH)
+  @Path(SERVICE_ROOT_PATH)
   @Consumes({ MediaType.APPLICATION_JSON })
   @Produces({ MediaType.APPLICATION_JSON })
-  public Response createApplication(Application application) {
-    LOG.info("POST: createApplication = {}", application);
-    ApplicationStatus applicationStatus = new ApplicationStatus();
+  public Response createService(Service service) {
+    LOG.info("POST: createService = {}", service);
+    ServiceStatus serviceStatus = new ServiceStatus();
     try {
-      ApplicationId applicationId = SERVICE_CLIENT.actionCreate(application);
-      LOG.info("Successfully created application " + application.getName()
+      ApplicationId applicationId = SERVICE_CLIENT.actionCreate(service);
+      LOG.info("Successfully created service " + service.getName()
           + " applicationId = " + applicationId);
-      applicationStatus.setState(ApplicationState.ACCEPTED);
-      applicationStatus.setUri(
-          CONTEXT_ROOT + APP_ROOT_PATH + "/" + application
+      serviceStatus.setState(ServiceState.ACCEPTED);
+      serviceStatus.setUri(
+          CONTEXT_ROOT + SERVICE_ROOT_PATH + "/" + service
               .getName());
-      return Response.status(Status.CREATED).entity(applicationStatus).build();
+      return Response.status(Status.CREATED).entity(serviceStatus).build();
     } catch (IllegalArgumentException e) {
-      applicationStatus.setDiagnostics(e.getMessage());
-      return Response.status(Status.BAD_REQUEST).entity(applicationStatus)
+      serviceStatus.setDiagnostics(e.getMessage());
+      return Response.status(Status.BAD_REQUEST).entity(serviceStatus)
           .build();
     } catch (Exception e) {
-      String message = "Failed to create application " + application.getName();
+      String message = "Failed to create service " + service.getName();
       LOG.error(message, e);
-      applicationStatus.setDiagnostics(message + ": " + e.getMessage());
+      serviceStatus.setDiagnostics(message + ": " + e.getMessage());
       return Response.status(Status.INTERNAL_SERVER_ERROR)
-          .entity(applicationStatus).build();
+          .entity(serviceStatus).build();
     }
   }
 
   @GET
-  @Path(APP_PATH)
+  @Path(SERVICE_PATH)
   @Consumes({ MediaType.APPLICATION_JSON })
   @Produces({ MediaType.APPLICATION_JSON })
-  public Response getApplication(@PathParam(APP_NAME) String appName) {
-    LOG.info("GET: getApplication for appName = {}", appName);
-    ApplicationStatus applicationStatus = new ApplicationStatus();
-
-    // app name validation
-    if (!SliderUtils.isClusternameValid(appName)) {
-      applicationStatus.setDiagnostics("Invalid application name: " + appName);
-      applicationStatus.setCode(ERROR_CODE_APP_NAME_INVALID);
-      return Response.status(Status.NOT_FOUND).entity(applicationStatus)
-          .build();
-    }
-
+  public Response getService(@PathParam(SERVICE_NAME) String appName) {
+    LOG.info("GET: getService for appName = {}", appName);
+    ServiceStatus serviceStatus = new ServiceStatus();
     try {
-      Application app = SERVICE_CLIENT.getStatus(appName);
+      Service app = SERVICE_CLIENT.getStatus(appName);
       return Response.ok(app).build();
+    } catch (IllegalArgumentException e) {
+      serviceStatus.setDiagnostics(e.getMessage());
+      serviceStatus.setCode(ERROR_CODE_APP_NAME_INVALID);
+      return Response.status(Status.NOT_FOUND).entity(serviceStatus)
+          .build();
     } catch (Exception e) {
-      LOG.error("Get application failed", e);
-      applicationStatus
-          .setDiagnostics("Failed to retrieve application: " + e.getMessage());
+      LOG.error("Get service failed", e);
+      serviceStatus
+          .setDiagnostics("Failed to retrieve service: " + e.getMessage());
       return Response.status(Status.INTERNAL_SERVER_ERROR)
-          .entity(applicationStatus).build();
+          .entity(serviceStatus).build();
     }
   }
 
   @DELETE
-  @Path(APP_PATH)
+  @Path(SERVICE_PATH)
   @Consumes({ MediaType.APPLICATION_JSON })
   @Produces({ MediaType.APPLICATION_JSON })
-  public Response deleteApplication(@PathParam(APP_NAME) String appName) {
-    LOG.info("DELETE: deleteApplication for appName = {}", appName);
-    return stopApplication(appName, true);
+  public Response deleteService(@PathParam(SERVICE_NAME) String appName) {
+    LOG.info("DELETE: deleteService for appName = {}", appName);
+    return stopService(appName, true);
   }
 
-  private Response stopApplication(String appName, boolean destroy) {
+  private Response stopService(String appName, boolean destroy) {
     try {
       SERVICE_CLIENT.actionStop(appName, destroy);
       if (destroy) {
         SERVICE_CLIENT.actionDestroy(appName);
-        LOG.info("Successfully deleted application {}", appName);
+        LOG.info("Successfully deleted service {}", appName);
       } else {
-        LOG.info("Successfully stopped application {}", appName);
+        LOG.info("Successfully stopped service {}", appName);
       }
       return Response.status(Status.NO_CONTENT).build();
     } catch (ApplicationNotFoundException e) {
-      ApplicationStatus applicationStatus = new ApplicationStatus();
-      applicationStatus.setDiagnostics(
-          "Application " + appName + " not found " + e.getMessage());
-      return Response.status(Status.NOT_FOUND).entity(applicationStatus)
+      ServiceStatus serviceStatus = new ServiceStatus();
+      serviceStatus.setDiagnostics(
+          "Service " + appName + " not found " + e.getMessage());
+      return Response.status(Status.NOT_FOUND).entity(serviceStatus)
           .build();
     } catch (Exception e) {
-      ApplicationStatus applicationStatus = new ApplicationStatus();
-      applicationStatus.setDiagnostics(e.getMessage());
+      ServiceStatus serviceStatus = new ServiceStatus();
+      serviceStatus.setDiagnostics(e.getMessage());
       return Response.status(Status.INTERNAL_SERVER_ERROR)
-          .entity(applicationStatus).build();
+          .entity(serviceStatus).build();
     }
   }
 
@@ -174,12 +173,12 @@ public class ApplicationApiService {
   @Path(COMPONENT_PATH)
   @Consumes({ MediaType.APPLICATION_JSON })
   @Produces({ MediaType.APPLICATION_JSON, MediaType.TEXT_PLAIN  })
-  public Response updateComponent(@PathParam(APP_NAME) String appName,
+  public Response updateComponent(@PathParam(SERVICE_NAME) String appName,
       @PathParam(COMPONENT_NAME) String componentName, Component component) {
 
     if (component.getNumberOfContainers() < 0) {
       return Response.status(Status.BAD_REQUEST).entity(
-          "Application = " + appName + ", Component = " + component.getName()
+          "Service = " + appName + ", Component = " + component.getName()
               + ": Invalid number of containers specified " + component
               .getNumberOfContainers()).build();
     }
@@ -192,7 +191,7 @@ public class ApplicationApiService {
               .get(componentName) + " to " + component.getNumberOfContainers())
           .build();
     } catch (YarnException | IOException e) {
-      ApplicationStatus status = new ApplicationStatus();
+      ServiceStatus status = new ServiceStatus();
       status.setDiagnostics(e.getMessage());
       return Response.status(Status.INTERNAL_SERVER_ERROR).entity(status)
           .build();
@@ -200,43 +199,43 @@ public class ApplicationApiService {
   }
 
   @PUT
-  @Path(APP_PATH)
+  @Path(SERVICE_PATH)
   @Consumes({ MediaType.APPLICATION_JSON })
   @Produces({ MediaType.APPLICATION_JSON })
-  public Response updateApplication(@PathParam(APP_NAME) String appName,
-      Application updateAppData) {
-    LOG.info("PUT: updateApplication for app = {} with data = {}", appName,
-        updateAppData);
+  public Response updateService(@PathParam(SERVICE_NAME) String appName,
+      Service updateServiceData) {
+    LOG.info("PUT: updateService for app = {} with data = {}", appName,
+        updateServiceData);
 
-    // Ignore the app name provided in updateAppData and always use appName
+    // Ignore the app name provided in updateServiceData and always use appName
     // path param
-    updateAppData.setName(appName);
+    updateServiceData.setName(appName);
 
     // For STOP the app should be running. If already stopped then this
     // operation will be a no-op. For START it should be in stopped state.
     // If already running then this operation will be a no-op.
-    if (updateAppData.getState() != null
-        && updateAppData.getState() == ApplicationState.STOPPED) {
-      return stopApplication(appName, false);
+    if (updateServiceData.getState() != null
+        && updateServiceData.getState() == ServiceState.STOPPED) {
+      return stopService(appName, false);
     }
 
     // If a START is requested
-    if (updateAppData.getState() != null
-        && updateAppData.getState() == ApplicationState.STARTED) {
-      return startApplication(appName);
+    if (updateServiceData.getState() != null
+        && updateServiceData.getState() == ServiceState.STARTED) {
+      return startService(appName);
     }
 
     // If new lifetime value specified then update it
-    if (updateAppData.getLifetime() != null
-        && updateAppData.getLifetime() > 0) {
-      return updateLifetime(appName, updateAppData);
+    if (updateServiceData.getLifetime() != null
+        && updateServiceData.getLifetime() > 0) {
+      return updateLifetime(appName, updateServiceData);
     }
 
     // flex a single component app
-    if (updateAppData.getNumberOfContainers() != null && !ServiceApiUtil
-        .hasComponent(updateAppData)) {
-      Component defaultComp = ServiceApiUtil.createDefaultComponent(updateAppData);
-      return updateComponent(updateAppData.getName(), defaultComp.getName(),
+    if (updateServiceData.getNumberOfContainers() != null && !ServiceApiUtil
+        .hasComponent(updateServiceData)) {
+      Component defaultComp = ServiceApiUtil.createDefaultComponent(updateServiceData);
+      return updateComponent(updateServiceData.getName(), defaultComp.getName(),
           defaultComp);
     }
 
@@ -244,15 +243,15 @@ public class ApplicationApiService {
     return Response.status(Status.NO_CONTENT).build();
   }
 
-  private Response updateLifetime(String appName, Application updateAppData) {
+  private Response updateLifetime(String appName, Service updateAppData) {
     try {
       String newLifeTime =
           SERVICE_CLIENT.updateLifetime(appName, updateAppData.getLifetime());
-      return Response.ok("Application " + appName + " lifeTime is successfully updated to "
+      return Response.ok("Service " + appName + " lifeTime is successfully updated to "
           + updateAppData.getLifetime() + " seconds from now: " + newLifeTime).build();
     } catch (Exception e) {
       String message =
-          "Failed to update application (" + appName + ") lifetime ("
+          "Failed to update service (" + appName + ") lifetime ("
               + updateAppData.getLifetime() + ")";
       LOG.error(message, e);
       return Response.status(Status.INTERNAL_SERVER_ERROR)
@@ -260,13 +259,13 @@ public class ApplicationApiService {
     }
   }
 
-  private Response startApplication(String appName) {
+  private Response startService(String appName) {
     try {
       SERVICE_CLIENT.actionStart(appName);
-      LOG.info("Successfully started application " + appName);
-      return Response.ok("Application " + appName + " is successfully started").build();
+      LOG.info("Successfully started service " + appName);
+      return Response.ok("Service " + appName + " is successfully started").build();
     } catch (Exception e) {
-      String message = "Failed to start application " + appName;
+      String message = "Failed to start service " + appName;
       LOG.info(message, e);
       return Response.status(Status.INTERNAL_SERVER_ERROR)
           .entity(message + ": " + e.getMessage()).build();

+ 159 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/java/org/apache/hadoop/yarn/service/webapp/ApiServerWebApp.java

@@ -0,0 +1,159 @@
+/*
+ * 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.service.webapp;
+
+import org.apache.hadoop.http.HttpServer2;
+import org.apache.hadoop.net.NetUtils;
+import org.apache.hadoop.security.AuthenticationFilterInitializer;
+import org.apache.hadoop.security.SecurityUtil;
+import org.apache.hadoop.security.UserGroupInformation;
+import org.apache.hadoop.service.AbstractService;
+import org.apache.hadoop.util.StringUtils;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
+import org.apache.hadoop.yarn.webapp.GenericExceptionHandler;
+import org.apache.hadoop.yarn.webapp.YarnJacksonJaxbJsonProvider;
+import org.eclipse.jetty.webapp.Configuration;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.net.InetSocketAddress;
+import java.net.URI;
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.List;
+
+import static org.apache.hadoop.yarn.conf.YarnConfiguration.RM_WEBAPP_SPNEGO_KEYTAB_FILE_KEY;
+import static org.apache.hadoop.yarn.conf.YarnConfiguration.RM_WEBAPP_SPNEGO_USER_NAME_KEY;
+import static org.apache.hadoop.yarn.service.conf.YarnServiceConf.*;
+
+/**
+ * This class launches the web service using Hadoop HttpServer2 (which uses
+ * an embedded Jetty container). This is the entry point to your service.
+ * The Java command used to launch this app should call the main method.
+ */
+public class ApiServerWebApp extends AbstractService {
+  private static final Logger logger = LoggerFactory
+      .getLogger(ApiServerWebApp.class);
+  private static final String SEP = ";";
+
+  // REST API server for YARN native services
+  private HttpServer2 apiServer;
+  private InetSocketAddress bindAddress;
+
+  public static void main(String[] args) throws IOException {
+    ApiServerWebApp apiWebApp = new ApiServerWebApp();
+    try {
+      apiWebApp.startWebApp();
+    } catch (Exception e) {
+      apiWebApp.close();
+    }
+  }
+
+  public ApiServerWebApp() {
+    super(ApiServerWebApp.class.getName());
+  }
+
+  @Override
+  protected void serviceStart() throws Exception {
+    bindAddress = getConfig().getSocketAddr(API_SERVER_ADDRESS,
+        DEFAULT_API_SERVER_ADDRESS , DEFAULT_API_SERVER_PORT);
+    logger.info("YARN API server running on " + bindAddress);
+    if (UserGroupInformation.isSecurityEnabled()) {
+      doSecureLogin(getConfig());
+    }
+    startWebApp();
+    super.serviceStart();
+  }
+
+  @Override
+  protected void serviceStop() throws Exception {
+    if (apiServer != null) {
+      apiServer.stop();
+    }
+    super.serviceStop();
+  }
+
+  private void doSecureLogin(org.apache.hadoop.conf.Configuration conf)
+      throws IOException {
+    SecurityUtil.login(conf, YarnConfiguration.RM_KEYTAB,
+        YarnConfiguration.RM_PRINCIPAL, bindAddress.getHostName());
+    addFilters(conf);
+  }
+
+  private void addFilters(org.apache.hadoop.conf.Configuration conf) {
+    // Always load pseudo authentication filter to parse "user.name" in an URL
+    // to identify a HTTP request's user.
+    boolean hasHadoopAuthFilterInitializer = false;
+    String filterInitializerConfKey = "hadoop.http.filter.initializers";
+    Class<?>[] initializersClasses =
+        conf.getClasses(filterInitializerConfKey);
+    List<String> targets = new ArrayList<String>();
+    if (initializersClasses != null) {
+      for (Class<?> initializer : initializersClasses) {
+        if (initializer.getName().equals(
+            AuthenticationFilterInitializer.class.getName())) {
+          hasHadoopAuthFilterInitializer = true;
+          break;
+        }
+        targets.add(initializer.getName());
+      }
+    }
+    if (!hasHadoopAuthFilterInitializer) {
+      targets.add(AuthenticationFilterInitializer.class.getName());
+      conf.set(filterInitializerConfKey, StringUtils.join(",", targets));
+    }
+  }
+
+  private void startWebApp() throws IOException {
+    URI uri = URI.create("http://" + NetUtils.getHostPortString(bindAddress));
+
+    apiServer = new HttpServer2.Builder()
+        .setName("api-server")
+        .setConf(getConfig())
+        .setSecurityEnabled(UserGroupInformation.isSecurityEnabled())
+        .setUsernameConfKey(RM_WEBAPP_SPNEGO_USER_NAME_KEY)
+        .setKeytabConfKey(RM_WEBAPP_SPNEGO_KEYTAB_FILE_KEY)
+        .addEndpoint(uri).build();
+
+    String apiPackages =
+        ApiServer.class.getPackage().getName() + SEP
+            + GenericExceptionHandler.class.getPackage().getName() + SEP
+            + YarnJacksonJaxbJsonProvider.class.getPackage().getName();
+    apiServer.addJerseyResourcePackage(apiPackages, "/*");
+
+    try {
+      logger.info("Service starting up. Logging start...");
+      apiServer.start();
+      logger.info("Server status = {}", apiServer.toString());
+      for (Configuration conf : apiServer.getWebAppContext()
+          .getConfigurations()) {
+        logger.info("Configurations = {}", conf);
+      }
+      logger.info("Context Path = {}", Collections.singletonList(
+          apiServer.getWebAppContext().getContextPath()));
+      logger.info("ResourceBase = {}", Collections.singletonList(
+          apiServer.getWebAppContext().getResourceBase()));
+      logger.info("War = {}", Collections
+          .singletonList(apiServer.getWebAppContext().getWar()));
+    } catch (Exception ex) {
+      logger.error("Hadoop HttpServer2 App **failed**", ex);
+      throw ex;
+    }
+  }
+}

+ 0 - 123
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/java/org/apache/hadoop/yarn/service/webapp/ApplicationApiWebApp.java

@@ -1,123 +0,0 @@
-/*
- * 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.service.webapp;
-
-import static org.apache.hadoop.yarn.service.conf.RestApiConstants.*;
-
-import java.io.IOException;
-import java.net.InetAddress;
-import java.net.URI;
-import java.util.Arrays;
-
-import org.apache.commons.lang.StringUtils;
-import org.apache.hadoop.http.HttpServer2;
-import org.apache.hadoop.service.AbstractService;
-import org.apache.hadoop.yarn.webapp.GenericExceptionHandler;
-import org.apache.hadoop.yarn.webapp.YarnJacksonJaxbJsonProvider;
-import org.eclipse.jetty.webapp.Configuration;
-import org.slf4j.Logger;
-import org.slf4j.LoggerFactory;
-
-/**
- * This class launches the web application using Hadoop HttpServer2 (which uses
- * an embedded Jetty container). This is the entry point to your application.
- * The Java command used to launch this app should call the main method.
- */
-public class ApplicationApiWebApp extends AbstractService {
-  private static final Logger logger = LoggerFactory
-      .getLogger(ApplicationApiWebApp.class);
-  private static final String SEP = ";";
-
-  // REST API server for YARN native services
-  private HttpServer2 applicationApiServer;
-
-  public static void main(String[] args) throws IOException {
-    ApplicationApiWebApp apiWebApp = new ApplicationApiWebApp();
-    try {
-      apiWebApp.startWebApp();
-    } catch (Exception e) {
-      if (apiWebApp != null) {
-        apiWebApp.close();
-      }
-    }
-  }
-
-  public ApplicationApiWebApp() {
-    super(ApplicationApiWebApp.class.getName());
-  }
-
-  @Override
-  protected void serviceStart() throws Exception {
-    startWebApp();
-    super.serviceStart();
-  }
-
-  @Override
-  protected void serviceStop() throws Exception {
-    if (applicationApiServer != null) {
-      applicationApiServer.stop();
-    }
-    super.serviceStop();
-  }
-
-  protected void startWebApp() throws IOException {
-    // The port that we should run on can be set into an environment variable
-    // Look for that variable and default to 9191 if it isn't there.
-    String webPort = System.getenv(PROPERTY_REST_SERVICE_PORT);
-    if (StringUtils.isEmpty(webPort)) {
-      webPort = "9191";
-    }
-
-    String webHost = System.getenv(PROPERTY_REST_SERVICE_HOST);
-    if (StringUtils.isEmpty(webHost)) {
-      webHost = InetAddress.getLocalHost().getHostName();
-    }
-    logger.info("YARN native services REST API running on host {} and port {}",
-        webHost, webPort);
-    logger.info("Configuration = {}", getConfig());
-
-    applicationApiServer = new HttpServer2.Builder()
-        .setName("services-rest-api")
-        .addEndpoint(URI.create("http://" + webHost + ":" + webPort)).build();
-
-    String apiPackages =
-        ApplicationApiService.class.getPackage().getName() + SEP
-            + GenericExceptionHandler.class.getPackage().getName() + SEP
-            + YarnJacksonJaxbJsonProvider.class.getPackage().getName();
-    applicationApiServer.addJerseyResourcePackage(apiPackages, "/*");
-
-    try {
-      logger.info("Application starting up. Logging start...");
-      applicationApiServer.start();
-      logger.info("Server status = {}", applicationApiServer.toString());
-      for (Configuration conf : applicationApiServer.getWebAppContext()
-          .getConfigurations()) {
-        logger.info("Configurations = {}", conf);
-      }
-      logger.info("Context Path = {}", Arrays.asList(applicationApiServer
-          .getWebAppContext().getContextPath()));
-      logger.info("ResourceBase = {}", Arrays.asList(applicationApiServer
-          .getWebAppContext().getResourceBase()));
-      logger.info("War = {}",
-          Arrays.asList(applicationApiServer.getWebAppContext().getWar()));
-    } catch (Exception ex) {
-      logger.error("Hadoop HttpServer2 App **failed**", ex);
-      throw ex;
-    }
-  }
-}

+ 93 - 92
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/resources/definition/YARN-Simplified-V1-API-Layer-For-Services.yaml

@@ -22,7 +22,7 @@ info:
     Bringing a new service on YARN today is not a simple experience. The APIs of existing frameworks are either too low level (native YARN), require writing new code (for frameworks with programmatic APIs) or writing a complex spec (for declarative frameworks). In addition to building critical building blocks inside YARN (as part of other efforts at link:https://issues.apache.org/jira/browse/YARN-4692[YARN-4692]), there is a need for simplifying the user facing story for building services. Experience of projects like Apache Slider running real-life services like HBase, Storm, Accumulo, Solr etc, gives us some very good insights on how simplified APIs for services should look like.
 
 
-    To this end, we should look at a new simple-services API layer backed by REST interfaces. This API can be used to create and manage the lifecycle of YARN services. Services here can range from simple single-component apps to complex multi-component assemblies needing orchestration.
+    To this end, we should look at a new simple-services API layer backed by REST interfaces. This API can be used to create and manage the lifecycle of YARN services. Services here can range from simple single-component service to complex multi-component assemblies needing orchestration.
 
 
     We should also look at making this a unified REST based entry point for other important features like resource-profile management (link:https://issues.apache.org/jira/browse/YARN-3926[YARN-3926]), package-definitions' lifecycle-management and service-discovery (link:https://issues.apache.org/jira/browse/YARN-913[YARN-913]/link:https://issues.apache.org/jira/browse/YARN-4757[YARN-4757]). We also need to flesh out its relation to our present much lower level REST APIs (link:https://issues.apache.org/jira/browse/YARN-1695[YARN-1695]) in YARN for application-submission and management.
@@ -41,177 +41,177 @@ schemes:
   - http
   - https
 # will be prefixed to all paths
-basePath: /services/v1/
+basePath: /ws/v1/
 consumes:
   - application/json
 produces:
   - application/json
 paths:
-  /applications:
+  /services:
     get:
-      summary: List of applications/services running in the cluster
-      description: Get a list of all currently running applications (response includes a minimal projection of the application info). For more details do a GET on a specific application name.
+      summary: List of services running in the cluster
+      description: Get a list of all currently running services (response includes a minimal projection of the service info). For more details do a GET on a specific service name.
       responses:
         200:
-          description: An array of applications
+          description: An array of services
           schema:
             type: array
             items:
-              $ref: '#/definitions/Application'
+              $ref: '#/definitions/Service'
         default:
           description: Unexpected error
           schema:
-            $ref: '#/definitions/ApplicationStatus'
+            $ref: '#/definitions/ServiceStatus'
     post:
-      summary: Create an application/service
-      description: Create an application. The request JSON is an Application object with details required for creation. If the request is successful it returns 202 Accepted. A success of this API only confirms success in submission of the application creation request. There is no guarantee that the application will actually reach a RUNNING state. Resource availability and several other factors determines if the application will be deployed in the cluster. It is expected that clients would subsequently call the GET API to get details of the application and determine its state.
+      summary: Create a service
+      description: Create a service. The request JSON is a service object with details required for creation. If the request is successful it returns 202 Accepted. A success of this API only confirms success in submission of the service creation request. There is no guarantee that the service will actually reach a RUNNING state. Resource availability and several other factors determines if the service will be deployed in the cluster. It is expected that clients would subsequently call the GET API to get details of the service and determine its state.
       parameters:
-        - name: Application
+        - name: Service
           in: body
-          description: Application request object
+          description: Service request object
           required: true
           schema:
-            $ref: '#/definitions/Application'
+            $ref: '#/definitions/Service'
       responses:
         202:
           description: Request accepted
         default:
           description: Unexpected error
           schema:
-            $ref: '#/definitions/ApplicationStatus'
+            $ref: '#/definitions/ServiceStatus'
 
-  /applications/{app_name}:
+  /services/{service_name}:
     get:
-      summary: Get an application/service details
-      description: Return the details (including containers) of a running application
+      summary: Get service details
+      description: Return the details (including containers) of a running service
       parameters:
-        - name: app_name
+        - name: service_name
           in: path
-          description: Application name
+          description: Service name
           required: true
           type: string
       responses:
         200:
-          description: An application object
+          description: a service object
           schema:
             type: object
             items:
-              $ref: '#/definitions/Application'
+              $ref: '#/definitions/Service'
           examples:
-            app_name: logsearch
+            service_name: logsearch
             artifact:
               id: logsearch:latest
               type: docker
         404:
-          description: Application does not exist
+          description: Service does not exist
         default:
           description: Unexpected error
           schema:
-            $ref: '#/definitions/ApplicationStatus'
+            $ref: '#/definitions/ServiceStatus'
     put:
-      summary: Update an application/service or upgrade the binary version of the components of a running application
-      description: Update the runtime properties of an application. As of now, only update of lifetime and number of instances (flexing) of the components of an application is supported. The PUT operation is also used to orchestrate an upgrade of the application containers to a newer version of their artifacts.
+      summary: Update a service or upgrade the binary version of the components of a running service
+      description: Update the runtime properties of a service. As of now, only update of lifetime and number of instances (flexing) of the components of a service is supported. The PUT operation is also used to orchestrate an upgrade of the service containers to a newer version of their artifacts.
       parameters:
-        - name: app_name
+        - name: service_name
           in: path
-          description: Application name
+          description: Service name
           required: true
           type: string
       responses:
         204:
           description: Update or upgrade was successful
         404:
-          description: Application does not exist
+          description: Service does not exist
         default:
           description: Unexpected error
           schema:
-            $ref: '#/definitions/ApplicationStatus'
+            $ref: '#/definitions/ServiceStatus'
     delete:
-      summary: Destroy application/service
-      description: Destroy an application and release all resources. This API might have to return JSON data providing location of logs, etc. Not finalized yet.
+      summary: Destroy service
+      description: Destroy a service and release all resources. This API might have to return JSON data providing location of logs, etc. Not finalized yet.
       parameters:
-        - name: app_name
+        - name: service_name
           in: path
-          description: Application name
+          description: Service name
           required: true
           type: string
       responses:
         204:
           description: Destroy was successful
         404:
-          description: Application does not exist
+          description: Service does not exist
         default:
           description: Unexpected error
           schema:
-            $ref: '#/definitions/ApplicationStatus'
+            $ref: '#/definitions/ServiceStatus'
 
 definitions:
-  Application:
-    description: An Application resource has the following attributes.
+  Service:
+    description: a service resource has the following attributes.
     required:
       - name
     properties:
       name:
         type: string
-        description: A unique application name. If Registry DNS is enabled, the max length is 63 characters.
+        description: A unique service name. If Registry DNS is enabled, the max length is 63 characters.
       id:
         type: string
-        description: A unique application id.
+        description: A unique service id.
       artifact:
-        description: Artifact of single-component applications.
+        description: Artifact of single-component service.
         $ref: '#/definitions/Artifact'
       resource:
-        description: Resource of single-component applications or the global default for multi-component applications. Mandatory if it is a single-component application and if cpus and memory are not specified at the Application level.
+        description: Resource of single-component service or the global default for multi-component services. Mandatory if it is a single-component service and if cpus and memory are not specified at the Service level.
         $ref: '#/definitions/Resource'
       launch_command:
         type: string
-        description: The custom launch command of an application component (optional). If not specified for applications with docker images say, it will default to the default start command of the image. If there is a single component in this application, you can specify this without the need to have a 'components' section.
+        description: The custom launch command of a service component (optional). If not specified for services with docker images say, it will default to the default start command of the image. If there is a single component in this service, you can specify this without the need to have a 'components' section.
       launch_time:
         type: string
         format: date
-        description: The time when the application was created, e.g. 2016-03-16T01:01:49.000Z.
+        description: The time when the service was created, e.g. 2016-03-16T01:01:49.000Z.
       number_of_containers:
         type: integer
         format: int64
-        description: Number of containers for each app-component in the application. Each app-component can further override this app-level global default.
+        description: Number of containers for each component in the service. Each component can further override this service-level global default.
       number_of_running_containers:
         type: integer
         format: int64
-        description: In get response this provides the total number of running containers for this application (across all components) at the time of request. Note, a subsequent request can return a different number as and when more containers get allocated until it reaches the total number of containers or if a flex request has been made between the two requests.
+        description: In get response this provides the total number of running containers for this service (across all components) at the time of request. Note, a subsequent request can return a different number as and when more containers get allocated until it reaches the total number of containers or if a flex request has been made between the two requests.
       lifetime:
         type: integer
         format: int64
-        description: Life time (in seconds) of the application from the time it reaches the STARTED state (after which it is automatically destroyed by YARN). For unlimited lifetime do not set a lifetime value.
+        description: Life time (in seconds) of the service from the time it reaches the STARTED state (after which it is automatically destroyed by YARN). For unlimited lifetime do not set a lifetime value.
       placement_policy:
-        description: Advanced scheduling and placement policies (optional). If not specified, it defaults to the default placement policy of the app owner. The design of placement policies are in the works. It is not very clear at this point, how policies in conjunction with labels be exposed to application owners. This is a placeholder for now. The advanced structure of this attribute will be determined by YARN-4902.
+        description: Advanced scheduling and placement policies (optional). If not specified, it defaults to the default placement policy of the service owner. The design of placement policies are in the works. It is not very clear at this point, how policies in conjunction with labels be exposed to service owners. This is a placeholder for now. The advanced structure of this attribute will be determined by YARN-4902.
         $ref: '#/definitions/PlacementPolicy'
       components:
-        description: Components of an application.
+        description: Components of a service.
         type: array
         items:
           $ref: '#/definitions/Component'
       configuration:
-        description: Config properties of an application. Configurations provided at the application/global level are available to all the components. Specific properties can be overridden at the component level.
+        description: Config properties of a service. Configurations provided at the service/global level are available to all the components. Specific properties can be overridden at the component level.
         $ref: '#/definitions/Configuration'
       containers:
-        description: Containers of a started application. Specifying a value for this attribute for the POST payload raises a validation error. This blob is available only in the GET response of a started application.
+        description: Containers of a started service. Specifying a value for this attribute for the POST payload raises a validation error. This blob is available only in the GET response of a started service.
         type: array
         items:
           $ref: '#/definitions/Container'
       state:
-        description: State of the application. Specifying a value for this attribute for the POST payload raises a validation error. This attribute is available only in the GET response of a started application.
-        $ref: '#/definitions/ApplicationState'
+        description: State of the service. Specifying a value for this attribute for the POST payload raises a validation error. This attribute is available only in the GET response of a started service.
+        $ref: '#/definitions/ServiceState'
       quicklinks:
         type: object
-        description: A blob of key-value pairs of quicklinks to be exported for an application.
+        description: A blob of key-value pairs of quicklinks to be exported for a service.
         additionalProperties:
           type: string
       queue:
         type: string
-        description: The YARN queue that this application should be submitted to.
+        description: The YARN queue that this service should be submitted to.
   Resource:
     description:
-      Resource determines the amount of resources (vcores, memory, network, etc.) usable by a container. This field determines the resource to be applied for all the containers of a component or application. The resource specified at the app (or global) level can be overriden at the component level. Only one of profile OR cpu & memory are exepected. It raises a validation exception otherwise.
+      Resource determines the amount of resources (vcores, memory, network, etc.) usable by a container. This field determines the resource to be applied for all the containers of a component or service. The resource specified at the service (or global) level can be overriden at the component level. Only one of profile OR cpu & memory are expected. It raises a validation exception otherwise.
     properties:
       profile:
         type: string
@@ -224,75 +224,75 @@ definitions:
         type: string
         description: Amount of memory allocated to each container (optional but overrides memory in profile if specified). Currently accepts only an integer value and default unit is in MB.
   PlacementPolicy:
-    description: Placement policy of an instance of an application. This feature is in the works in YARN-4902.
+    description: Placement policy of an instance of a service. This feature is in the works in YARN-6592.
     properties:
       label:
         type: string
-        description: Assigns an app to a named partition of the cluster where the application desires to run (optional). If not specified all apps are submitted to a default label of the app owner. One or more labels can be setup for each application owner account with required constraints like no-preemption, sla-99999, preemption-ok, etc.
+        description: Assigns a service to a named partition of the cluster where the service desires to run (optional). If not specified all services are submitted to a default label of the service owner. One or more labels can be setup for each service owner account with required constraints like no-preemption, sla-99999, preemption-ok, etc.
   Artifact:
-    description: Artifact of an application component. If not specified, component will just run the bare launch command and no artifact will be localized.
+    description: Artifact of a service component. If not specified, component will just run the bare launch command and no artifact will be localized.
     required:
     - id
     properties:
       id:
         type: string
-        description: Artifact id. Examples are package location uri for tarball based apps, image name for docker, name of application, etc.
+        description: Artifact id. Examples are package location uri for tarball based services, image name for docker, name of service, etc.
       type:
         type: string
-        description: Artifact type, like docker, tarball, etc. (optional). For TARBALL type, the specified tarball will be localized to the container local working directory under a folder named lib. For APPLICATION type, the application specified will be read and its components will be added into this application. The original component with artifact type APPLICATION will be removed (any properties specified in the original component will be ignored).
+        description: Artifact type, like docker, tarball, etc. (optional). For TARBALL type, the specified tarball will be localized to the container local working directory under a folder named lib. For SERVICE type, the service specified will be read and its components will be added into this service. The original component with artifact type SERVICE will be removed (any properties specified in the original component will be ignored).
         enum:
           - DOCKER
           - TARBALL
-          - APPLICATION
+          - SERVICE
         default: DOCKER
       uri:
         type: string
         description: Artifact location to support multiple artifact stores (optional).
   Component:
-    description: One or more components of the application. If the application is HBase say, then the component can be a simple role like master or regionserver. If the application is a complex business webapp then a component can be other applications say Kafka or Storm. Thereby it opens up the support for complex and nested applications.
+    description: One or more components of the service. If the service is HBase say, then the component can be a simple role like master or regionserver. If the service is a complex business webapp then a component can be other services say Kafka or Storm. Thereby it opens up the support for complex and nested services.
     required:
     - name
     properties:
       name:
         type: string
-        description: Name of the application component (mandatory). If Registry DNS is enabled, the max length is 63 characters. If unique component support is enabled, the max length is lowered to 44 characters.
+        description: Name of the service component (mandatory). If Registry DNS is enabled, the max length is 63 characters. If unique component support is enabled, the max length is lowered to 44 characters.
       dependencies:
         type: array
         items:
           type: string
-        description: An array of application components which should be in READY state (as defined by readiness check), before this component can be started. The dependencies across all components of an application should be represented as a DAG.
+        description: An array of service components which should be in READY state (as defined by readiness check), before this component can be started. The dependencies across all components of a service should be represented as a DAG.
       readiness_check:
-        description: Readiness check for this app-component.
+        description: Readiness check for this component.
         $ref: '#/definitions/ReadinessCheck'
       artifact:
-        description: Artifact of the component (optional). If not specified, the application level global artifact takes effect.
+        description: Artifact of the component (optional). If not specified, the service level global artifact takes effect.
         $ref: '#/definitions/Artifact'
       launch_command:
         type: string
         description: The custom launch command of this component (optional for DOCKER component, required otherwise). When specified at the component level, it overrides the value specified at the global level (if any).
       resource:
-        description: Resource of this component (optional). If not specified, the application level global resource takes effect.
+        description: Resource of this component (optional). If not specified, the service level global resource takes effect.
         $ref: '#/definitions/Resource'
       number_of_containers:
         type: integer
         format: int64
-        description: Number of containers for this app-component (optional). If not specified, the application level global number_of_containers takes effect.
+        description: Number of containers for this component (optional). If not specified, the service level global number_of_containers takes effect.
       run_privileged_container:
         type: boolean
         description: Run all containers of this component in privileged mode (YARN-4262).
       placement_policy:
-        description: Advanced scheduling and placement policies for all containers of this component (optional). If not specified, the app level placement_policy takes effect. Refer to the description at the global level for more details.
+        description: Advanced scheduling and placement policies for all containers of this component (optional). If not specified, the service level placement_policy takes effect. Refer to the description at the global level for more details.
         $ref: '#/definitions/PlacementPolicy'
       configuration:
-        description: Config properties for this app-component.
+        description: Config properties for this component.
         $ref: '#/definitions/Configuration'
       quicklinks:
         type: array
         items:
           type: string
-        description: A list of quicklink keys defined at the application level, and to be resolved by this component.
+        description: A list of quicklink keys defined at the service level, and to be resolved by this component.
   ReadinessCheck:
-    description: A custom command or a pluggable helper container to determine the readiness of a container of a component. Readiness for every application is different. Hence the need for a simple interface, with scope to support advanced usecases.
+    description: A custom command or a pluggable helper container to determine the readiness of a container of a component. Readiness for every service is different. Hence the need for a simple interface, with scope to support advanced usecases.
     required:
     - type
     properties:
@@ -308,28 +308,28 @@ definitions:
         additionalProperties:
           type: string
       artifact:
-        description: Artifact of the pluggable readiness check helper container (optional). If specified, this helper container typically hosts the http uri and encapsulates the complex scripts required to perform actual container readiness check. At the end it is expected to respond a 204 No content just like the simplified use case. This pluggable framework benefits application owners who can run applications without any packaging modifications. Note, artifacts of type docker only is supported for now. NOT IMPLEMENTED YET
+        description: Artifact of the pluggable readiness check helper container (optional). If specified, this helper container typically hosts the http uri and encapsulates the complex scripts required to perform actual container readiness check. At the end it is expected to respond a 204 No content just like the simplified use case. This pluggable framework benefits service owners who can run services without any packaging modifications. Note, artifacts of type docker only is supported for now. NOT IMPLEMENTED YET
         $ref: '#/definitions/Artifact'
   Configuration:
-    description: Set of configuration properties that can be injected into the application components via envs, files and custom pluggable helper docker containers. Files of several standard formats like xml, properties, json, yaml and templates will be supported.
+    description: Set of configuration properties that can be injected into the service components via envs, files and custom pluggable helper docker containers. Files of several standard formats like xml, properties, json, yaml and templates will be supported.
     properties:
       properties:
         type: object
-        description: A blob of key-value pairs of common application properties.
+        description: A blob of key-value pairs of common service properties.
         additionalProperties:
           type: string
       env:
         type: object
-        description: A blob of key-value pairs which will be appended to the default system properties and handed off to the application at start time. All placeholder references to properties will be substituted before injection.
+        description: A blob of key-value pairs which will be appended to the default system properties and handed off to the service at start time. All placeholder references to properties will be substituted before injection.
         additionalProperties:
           type: string
       files:
-        description: Array of list of files that needs to be created and made available as volumes in the application component containers.
+        description: Array of list of files that needs to be created and made available as volumes in the service component containers.
         type: array
         items:
           $ref: '#/definitions/ConfigFile'
   ConfigFile:
-    description: A config file that needs to be created and made available as a volume in an application component container.
+    description: A config file that needs to be created and made available as a volume in a service component container.
     properties:
       type:
         type: string
@@ -352,11 +352,11 @@ definitions:
         type: object
         description: A blob of key value pairs that will be dumped in the dest_file in the format as specified in type. If src_file is specified, src_file content are dumped in the dest_file and these properties will overwrite, if any, existing properties in src_file or be added as new properties in src_file.
   Container:
-    description: An instance of a running application container.
+    description: An instance of a running service container.
     properties:
       id:
         type: string
-        description: Unique container id of a running application, e.g. container_e3751_1458061340047_0008_01_000002.
+        description: Unique container id of a running service, e.g. container_e3751_1458061340047_0008_01_000002.
       launch_time:
         type: string
         format: date
@@ -371,7 +371,7 @@ definitions:
         type: string
         description: The bare node or host in which the container is running, e.g. cn008.example.com.
       state:
-        description: State of the container of an application.
+        description: State of the container of a service.
         $ref: '#/definitions/ContainerState'
       component_name:
         type: string
@@ -385,12 +385,12 @@ definitions:
       privileged_container:
         type: boolean
         description: Container running in privileged mode or not.
-  ApplicationState:
-    description: The current state of an application.
+  ServiceState:
+    description: The current state of a service.
     properties:
       state:
         type: string
-        description: enum of the state of the application
+        description: enum of the state of the service
         enum:
           - ACCEPTED
           - STARTED
@@ -398,25 +398,26 @@ definitions:
           - STOPPED
           - FAILED
   ContainerState:
-    description: The current state of the container of an application.
+    description: The current state of the container of a service.
     properties:
       state:
         type: string
         description: enum of the state of the container
         enum:
           - INIT
+          - STARTED
           - READY
-  ApplicationStatus:
-    description: The current status of a submitted application, returned as a response to the GET API.
+  ServiceStatus:
+    description: The current status of a submitted service, returned as a response to the GET API.
     properties:
       diagnostics:
         type: string
-        description: Diagnostic information (if any) for the reason of the current state of the application. It typically has a non-null value, if the application is in a non-running state.
+        description: Diagnostic information (if any) for the reason of the current state of the service. It typically has a non-null value, if the service is in a non-running state.
       state:
-        description: Application state.
-        $ref: '#/definitions/ApplicationState'
+        description: Service state.
+        $ref: '#/definitions/ServiceState'
       code:
         type: integer
         format: int32
-        description: An error code specific to a scenario which app owners should be able to use to understand the failure in addition to the diagnostic information.
+        description: An error code specific to a scenario which service owners should be able to use to understand the failure in addition to the diagnostic information.
 

+ 0 - 28
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services-api/src/main/scripts/run_rest_service.sh

@@ -1,28 +0,0 @@
-#!/usr/bin/env bash
-
-# 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.
-
-export SLIDER_VERSION=${project.version}
-export HDP_VERSION=${HDP_VERSION:-$SLIDER_VERSION}
-export SCRIPT_DIR="$( cd "$( dirname "$0" )" && pwd )"
-export LIB_PARENT_DIR=`dirname $SCRIPT_DIR`
-export JAVA_HOME=${JAVA_HOME:-/usr/jdk64/jdk1.8.0_40}
-export HADOOP_CONF_DIR=${HADOOP_CONF_DIR:-/etc/hadoop/conf}
-export REST_SERVICE_PORT=${REST_SERVICE_PORT:-9191}
-export APP_RUNAS_USER=${APP_RUNAS_USER:-root}
-export REST_SERVICE_LOG_DIR=${REST_SERVICE_LOG_DIR:-/tmp/}
-export JAVA_OPTS="-Xms256m -Xmx1024m -XX:+PrintGC -Xloggc:$REST_SERVICE_LOG_DIR/gc.log"
-$JAVA_HOME/bin/java $JAVA_OPTS -cp .:$HADOOP_CONF_DIR:$LIB_PARENT_DIR/services-api/*:$LIB_PARENT_DIR/slider/* -DREST_SERVICE_LOG_DIR=$REST_SERVICE_LOG_DIR -Dlog4j.configuration=log4j-server.properties -Dslider.libdir=$LIB_PARENT_DIR/slider org.apache.hadoop.yarn.services.webapp.ApplicationApiWebApp 1>>$REST_SERVICE_LOG_DIR/restservice-out.log 2>&1

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/dev-support/findbugs-exclude.xml → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/dev-support/findbugs-exclude.xml


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/conf/yarnservice-log4j.properties → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/conf/yarnservice-log4j.properties


+ 3 - 4
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/pom.xml → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/pom.xml

@@ -18,13 +18,12 @@
   <modelVersion>4.0.0</modelVersion>
   <parent>
     <groupId>org.apache.hadoop</groupId>
-    <artifactId>hadoop-yarn-slider</artifactId>
+    <artifactId>hadoop-yarn-services</artifactId>
     <version>3.0.0-beta1-SNAPSHOT</version>
   </parent>
-  <groupId>org.apache.hadoop</groupId>
-  <artifactId>hadoop-yarn-slider-core</artifactId>
+  <artifactId>hadoop-yarn-services-core</artifactId>
   <packaging>jar</packaging>
-  <name>Apache Hadoop YARN Slider Core</name>
+  <name>Apache Hadoop YARN Services Core</name>
 
   <properties>
     <!-- Needed for generating FindBugs warnings using parent pom -->

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/ClientAMProtocol.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ClientAMProtocol.java


+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/ClientAMService.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ClientAMService.java

@@ -101,7 +101,7 @@ public class ClientAMService extends AbstractService
   @Override
   public GetStatusResponseProto getStatus(GetStatusRequestProto request)
       throws IOException, YarnException {
-    String stat = ServiceApiUtil.jsonSerDeser.toJson(context.application);
+    String stat = ServiceApiUtil.jsonSerDeser.toJson(context.service);
     return GetStatusResponseProto.newBuilder().setStatus(stat).build();
   }
 

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/ContainerFailureTracker.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ContainerFailureTracker.java


+ 2 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/ServiceContext.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ServiceContext.java

@@ -21,12 +21,12 @@ package org.apache.hadoop.yarn.service;
 import com.google.common.cache.LoadingCache;
 import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
 import org.apache.hadoop.yarn.security.client.ClientToAMTokenSecretManager;
-import org.apache.hadoop.yarn.service.api.records.Application;
+import org.apache.hadoop.yarn.service.api.records.Service;
 import org.apache.hadoop.yarn.service.api.records.ConfigFile;
 import org.apache.hadoop.yarn.service.utils.SliderFileSystem;
 
 public class ServiceContext {
-  public Application application = null;
+  public Service service = null;
   public SliderFileSystem fs;
   public String serviceHdfsDir = "";
   public ApplicationAttemptId attemptId;

+ 5 - 6
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/ServiceMaster.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ServiceMaster.java

@@ -35,8 +35,7 @@ import org.apache.hadoop.yarn.conf.YarnConfiguration;
 import org.apache.hadoop.yarn.exceptions.YarnException;
 import org.apache.hadoop.yarn.security.client.ClientToAMTokenSecretManager;
 import org.apache.hadoop.yarn.service.client.params.SliderAMArgs;
-import org.apache.hadoop.yarn.service.exceptions.BadCommandArgumentsException;
-import org.apache.hadoop.yarn.service.servicemonitor.ServiceMonitor;
+import org.apache.hadoop.yarn.service.monitor.ServiceMonitor;
 import org.apache.hadoop.yarn.service.utils.ServiceApiUtil;
 import org.apache.hadoop.yarn.service.utils.SliderFileSystem;
 import org.apache.hadoop.yarn.service.utils.SliderUtils;
@@ -79,7 +78,7 @@ public class ServiceMaster extends CompositeService {
     ContainerId amContainerId = getAMContainerId();
 
     ApplicationAttemptId attemptId = amContainerId.getApplicationAttemptId();
-    LOG.info("Application attemptId: " + attemptId);
+    LOG.info("Service AppAttemptId: " + attemptId);
     context.attemptId = attemptId;
 
     // configure AM to wait forever for RM
@@ -119,9 +118,9 @@ public class ServiceMaster extends CompositeService {
 
   protected void loadApplicationJson(ServiceContext context,
       SliderFileSystem fs) throws IOException {
-    context.application = ServiceApiUtil
-        .loadApplicationFrom(fs, new Path(amArgs.getAppDefPath()));
-    LOG.info(context.application.toString());
+    context.service = ServiceApiUtil
+        .loadServiceFrom(fs, new Path(amArgs.getAppDefPath()));
+    LOG.info(context.service.toString());
   }
 
   @Override

+ 1 - 4
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/metrics/ServiceMetrics.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ServiceMetrics.java

@@ -16,7 +16,7 @@
  * limitations under the License.
  */
 
-package org.apache.hadoop.yarn.service.metrics;
+package org.apache.hadoop.yarn.service;
 
 import org.apache.hadoop.metrics2.MetricsCollector;
 import org.apache.hadoop.metrics2.MetricsInfo;
@@ -56,9 +56,6 @@ public class ServiceMetrics implements MetricsSource {
   @Metric("containers preempted")
   public MutableGaugeInt containersPreempted;
 
-  @Metric("containers exceeded limits")
-  public MutableGaugeInt containersLimitsExceeded;
-
   @Metric("containers surplus")
   public MutableGaugeInt surplusContainers;
 

+ 15 - 16
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/ServiceScheduler.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/ServiceScheduler.java

@@ -54,18 +54,17 @@ import org.apache.hadoop.yarn.event.AsyncDispatcher;
 import org.apache.hadoop.yarn.event.EventHandler;
 import org.apache.hadoop.yarn.exceptions.YarnException;
 import org.apache.hadoop.yarn.exceptions.YarnRuntimeException;
-import org.apache.hadoop.yarn.service.api.constants.ServiceApiConstants;
-import org.apache.hadoop.yarn.service.api.records.Application;
+import org.apache.hadoop.yarn.service.api.ServiceApiConstants;
+import org.apache.hadoop.yarn.service.api.records.Service;
 import org.apache.hadoop.yarn.service.api.records.ConfigFile;
-import org.apache.hadoop.yarn.service.compinstance.ComponentInstance;
-import org.apache.hadoop.yarn.service.compinstance.ComponentInstanceEvent;
-import org.apache.hadoop.yarn.service.compinstance.ComponentInstanceEventType;
+import org.apache.hadoop.yarn.service.component.instance.ComponentInstance;
+import org.apache.hadoop.yarn.service.component.instance.ComponentInstanceEvent;
+import org.apache.hadoop.yarn.service.component.instance.ComponentInstanceEventType;
 import org.apache.hadoop.yarn.service.component.Component;
 import org.apache.hadoop.yarn.service.component.ComponentEvent;
 import org.apache.hadoop.yarn.service.component.ComponentEventType;
 import org.apache.hadoop.yarn.service.conf.YarnServiceConstants;
 import org.apache.hadoop.yarn.service.containerlaunch.ContainerLaunchService;
-import org.apache.hadoop.yarn.service.metrics.ServiceMetrics;
 import org.apache.hadoop.yarn.service.provider.ProviderUtils;
 import org.apache.hadoop.yarn.service.registry.YarnRegistryViewForProviders;
 import org.apache.hadoop.yarn.service.timelineservice.ServiceMetricsSink;
@@ -92,7 +91,7 @@ import java.util.concurrent.TimeUnit;
 
 import static org.apache.hadoop.fs.FileSystem.FS_DEFAULT_NAME_KEY;
 import static org.apache.hadoop.registry.client.api.RegistryConstants.*;
-import static org.apache.hadoop.yarn.service.api.constants.ServiceApiConstants.*;
+import static org.apache.hadoop.yarn.service.api.ServiceApiConstants.*;
 import static org.apache.hadoop.yarn.service.component.ComponentEventType.*;
 
 /**
@@ -102,7 +101,7 @@ public class ServiceScheduler extends CompositeService {
 
   private static final Logger LOG =
       LoggerFactory.getLogger(ServiceScheduler.class);
-  private Application app;
+  private Service app;
 
   // component_name -> component
   private final Map<String, Component> componentsByName =
@@ -139,13 +138,13 @@ public class ServiceScheduler extends CompositeService {
   private ContainerLaunchService containerLaunchService;
 
   public ServiceScheduler(ServiceContext context) {
-    super(context.application.getName());
+    super(context.service.getName());
     this.context = context;
   }
 
   public void buildInstance(ServiceContext context, Configuration configuration)
       throws YarnException {
-    app = context.application;
+    app = context.service;
     executorService = Executors.newScheduledThreadPool(10);
     RegistryOperations registryClient = RegistryOperationsFactory
         .createInstance("ServiceScheduler", configuration);
@@ -157,7 +156,7 @@ public class ServiceScheduler extends CompositeService {
     serviceMetrics = ServiceMetrics
         .register(app.getName(), "Metrics for service");
     serviceMetrics.tag("type", "Metrics type [component or service]", "service");
-    serviceMetrics.tag("appId", "Application id for service", app.getId());
+    serviceMetrics.tag("appId", "Service id for service", app.getId());
 
     amRMClient = createAMRMClient();
     addIfService(amRMClient);
@@ -247,7 +246,7 @@ public class ServiceScheduler extends CompositeService {
         + "Navigate to the failed component for more details.";
     amRMClient
         .unregisterApplicationMaster(FinalApplicationStatus.ENDED, msg, "");
-    LOG.info("Application " + app.getName()
+    LOG.info("Service " + app.getName()
         + " unregistered with RM, with attemptId = " + context.attemptId
         + ", diagnostics = " + diagnostics);
     super.serviceStop();
@@ -337,8 +336,8 @@ public class ServiceScheduler extends CompositeService {
   }
 
   private void registerServiceInstance(ApplicationAttemptId attemptId,
-      Application application) throws IOException {
-    LOG.info("Registering " + attemptId + ", " + application.getName()
+      Service service) throws IOException {
+    LOG.info("Registering " + attemptId + ", " + service.getName()
         + " into registry");
     ServiceRecord serviceRecord = new ServiceRecord();
     serviceRecord.set(YarnRegistryAttributes.YARN_ID,
@@ -352,7 +351,7 @@ public class ServiceScheduler extends CompositeService {
             context.clientAMService.getBindAddress()));
 
     // set any provided attributes
-    setUserProvidedServiceRecordAttributes(application.getConfiguration(),
+    setUserProvidedServiceRecordAttributes(service.getConfiguration(),
         serviceRecord);
 
     executorService.submit(new Runnable() {
@@ -641,7 +640,7 @@ public class ServiceScheduler extends CompositeService {
     return componentsByName;
   }
 
-  public Application getApp() {
+  public Service getApp() {
     return app;
   }
 

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/constants/ServiceApiConstants.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/ServiceApiConstants.java

@@ -16,7 +16,7 @@
  * limitations under the License.
  */
 
-package org.apache.hadoop.yarn.service.api.constants;
+package org.apache.hadoop.yarn.service.api;
 
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;

+ 5 - 5
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Artifact.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Artifact.java

@@ -30,11 +30,11 @@ import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 
 /**
- * Artifact of an application component.
+ * Artifact of an service component.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "Artifact of an application component")
+@ApiModel(description = "Artifact of an service component")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 @JsonInclude(JsonInclude.Include.NON_NULL)
 public class Artifact implements Serializable {
@@ -43,7 +43,7 @@ public class Artifact implements Serializable {
   private String id = null;
 
   public enum TypeEnum {
-    DOCKER("DOCKER"), TARBALL("TARBALL"), APPLICATION("APPLICATION");
+    DOCKER("DOCKER"), TARBALL("TARBALL"), SERVICE("SERVICE");
 
     private String value;
 
@@ -62,7 +62,7 @@ public class Artifact implements Serializable {
   private String uri = null;
 
   /**
-   * Artifact id. Examples are package location uri for tarball based apps,
+   * Artifact id. Examples are package location uri for tarball based services,
    * image name for docker, etc.
    **/
   public Artifact id(String id) {
@@ -70,7 +70,7 @@ public class Artifact implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", required = true, value = "Artifact id. Examples are package location uri for tarball based apps, image name for docker, etc.")
+  @ApiModelProperty(example = "null", required = true, value = "Artifact id. Examples are package location uri for tarball based services, image name for docker, etc.")
   @JsonProperty("id")
   public String getId() {
     return id;

+ 3 - 3
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/BaseResource.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/BaseResource.java

@@ -29,9 +29,9 @@ public class BaseResource implements Serializable {
   private String uri;
 
   /**
-   * Resource location, e.g. \
-   * "/applications/helloworld/containers/container_e3751_1458061340047_0008_01_000002\
-   * "
+   * Resource location for a service, e.g.
+   * /ws/v1/services/helloworld
+   *
    **/
   public String getUri() {
     return uri;

+ 26 - 26
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Component.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Component.java

@@ -35,15 +35,15 @@ import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 
 /**
- * One or more components of the application. If the application is HBase say,
+ * One or more components of the service. If the service is HBase say,
  * then the component can be a simple role like master or regionserver. If the
- * application is a complex business webapp then a component can be other
- * applications say Kafka or Storm. Thereby it opens up the support for complex
- * and nested applications.
+ * service is a complex business webapp then a component can be other
+ * services say Kafka or Storm. Thereby it opens up the support for complex
+ * and nested services.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "One or more components of the application. If the application is HBase say, then the component can be a simple role like master or regionserver. If the application is a complex business webapp then a component can be other applications say Kafka or Storm. Thereby it opens up the support for complex and nested applications.")
+@ApiModel(description = "One or more components of the service. If the service is HBase say, then the component can be a simple role like master or regionserver. If the service is a complex business webapp then a component can be other services say Kafka or Storm. Thereby it opens up the support for complex and nested services.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 @XmlRootElement
 @JsonInclude(JsonInclude.Include.NON_NULL)
@@ -65,14 +65,14 @@ public class Component implements Serializable {
       Collections.synchronizedList(new ArrayList<Container>());
 
   /**
-   * Name of the application component (mandatory).
+   * Name of the service component (mandatory).
    **/
   public Component name(String name) {
     this.name = name;
     return this;
   }
 
-  @ApiModelProperty(example = "null", required = true, value = "Name of the application component (mandatory).")
+  @ApiModelProperty(example = "null", required = true, value = "Name of the service component (mandatory).")
   @JsonProperty("name")
   public String getName() {
     return name;
@@ -83,9 +83,9 @@ public class Component implements Serializable {
   }
 
   /**
-   * An array of application components which should be in READY state (as
+   * An array of service components which should be in READY state (as
    * defined by readiness check), before this component can be started. The
-   * dependencies across all components of an application should be represented
+   * dependencies across all components of a service should be represented
    * as a DAG.
    **/
   public Component dependencies(List<String> dependencies) {
@@ -93,7 +93,7 @@ public class Component implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "An array of application components which should be in READY state (as defined by readiness check), before this component can be started. The dependencies across all components of an application should be represented as a DAG.")
+  @ApiModelProperty(example = "null", value = "An array of service components which should be in READY state (as defined by readiness check), before this component can be started. The dependencies across all components of an service should be represented as a DAG.")
   @JsonProperty("dependencies")
   public List<String> getDependencies() {
     return dependencies;
@@ -104,14 +104,14 @@ public class Component implements Serializable {
   }
 
   /**
-   * Readiness check for this app-component.
+   * Readiness check for this component.
    **/
   public Component readinessCheck(ReadinessCheck readinessCheck) {
     this.readinessCheck = readinessCheck;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Readiness check for this app-component.")
+  @ApiModelProperty(example = "null", value = "Readiness check for this component.")
   @JsonProperty("readiness_check")
   public ReadinessCheck getReadinessCheck() {
     return readinessCheck;
@@ -123,7 +123,7 @@ public class Component implements Serializable {
   }
 
   /**
-   * Artifact of the component (optional). If not specified, the application
+   * Artifact of the component (optional). If not specified, the service
    * level global artifact takes effect.
    **/
   public Component artifact(Artifact artifact) {
@@ -131,7 +131,7 @@ public class Component implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Artifact of the component (optional). If not specified, the application level global artifact takes effect.")
+  @ApiModelProperty(example = "null", value = "Artifact of the component (optional). If not specified, the service level global artifact takes effect.")
   @JsonProperty("artifact")
   public Artifact getArtifact() {
     return artifact;
@@ -163,7 +163,7 @@ public class Component implements Serializable {
   }
 
   /**
-   * Resource of this component (optional). If not specified, the application
+   * Resource of this component (optional). If not specified, the service
    * level global resource takes effect.
    **/
   public Component resource(Resource resource) {
@@ -171,7 +171,7 @@ public class Component implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Resource of this component (optional). If not specified, the application level global resource takes effect.")
+  @ApiModelProperty(example = "null", value = "Resource of this component (optional). If not specified, the service level global resource takes effect.")
   @JsonProperty("resource")
   public Resource getResource() {
     return resource;
@@ -182,15 +182,15 @@ public class Component implements Serializable {
   }
 
   /**
-   * Number of containers for this app-component (optional). If not specified,
-   * the application level global number_of_containers takes effect.
+   * Number of containers for this component (optional). If not specified,
+   * the service level global number_of_containers takes effect.
    **/
   public Component numberOfContainers(Long numberOfContainers) {
     this.numberOfContainers = numberOfContainers;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Number of containers for this app-component (optional). If not specified, the application level global number_of_containers takes effect.")
+  @ApiModelProperty(example = "null", value = "Number of containers for this component (optional). If not specified, the service level global number_of_containers takes effect.")
   @JsonProperty("number_of_containers")
   public Long getNumberOfContainers() {
     return numberOfContainers;
@@ -201,7 +201,7 @@ public class Component implements Serializable {
     this.numberOfContainers = numberOfContainers;
   }
 
-  @ApiModelProperty(example = "null", value = "Containers of a started component. Specifying a value for this attribute for the POST payload raises a validation error. This blob is available only in the GET response of a started application.")
+  @ApiModelProperty(example = "null", value = "Containers of a started component. Specifying a value for this attribute for the POST payload raises a validation error. This blob is available only in the GET response of a started service.")
   @JsonProperty("containers")
   public List<Container> getContainers() {
     return containers;
@@ -248,7 +248,7 @@ public class Component implements Serializable {
 
   /**
    * Advanced scheduling and placement policies for all containers of this
-   * component (optional). If not specified, the app level placement_policy
+   * component (optional). If not specified, the service level placement_policy
    * takes effect. Refer to the description at the global level for more
    * details.
    **/
@@ -257,7 +257,7 @@ public class Component implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Advanced scheduling and placement policies for all containers of this component (optional). If not specified, the app level placement_policy takes effect. Refer to the description at the global level for more details.")
+  @ApiModelProperty(example = "null", value = "Advanced scheduling and placement policies for all containers of this component (optional). If not specified, the service level placement_policy takes effect. Refer to the description at the global level for more details.")
   @JsonProperty("placement_policy")
   public PlacementPolicy getPlacementPolicy() {
     return placementPolicy;
@@ -269,14 +269,14 @@ public class Component implements Serializable {
   }
 
   /**
-   * Config properties for this app-component.
+   * Config properties for this component.
    **/
   public Component configuration(Configuration configuration) {
     this.configuration = configuration;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Config properties for this app-component.")
+  @ApiModelProperty(example = "null", value = "Config properties for this component.")
   @JsonProperty("configuration")
   public Configuration getConfiguration() {
     return configuration;
@@ -287,7 +287,7 @@ public class Component implements Serializable {
   }
 
   /**
-   * A list of quicklink keys defined at the application level, and to be
+   * A list of quicklink keys defined at the service level, and to be
    * resolved by this component.
    **/
   public Component quicklinks(List<String> quicklinks) {
@@ -295,7 +295,7 @@ public class Component implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "A list of quicklink keys defined at the application level, and to be resolved by this component.")
+  @ApiModelProperty(example = "null", value = "A list of quicklink keys defined at the service level, and to be resolved by this component.")
   @JsonProperty("quicklinks")
   public List<String> getQuicklinks() {
     return quicklinks;

+ 4 - 4
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ConfigFile.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ConfigFile.java

@@ -34,11 +34,11 @@ import java.util.Objects;
 
 /**
  * A config file that needs to be created and made available as a volume in an
- * application component container.
+ * service component container.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "A config file that needs to be created and made available as a volume in an application component container.")
+@ApiModel(description = "A config file that needs to be created and made available as a volume in an service component container.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 @XmlRootElement
 @JsonInclude(JsonInclude.Include.NON_NULL)
@@ -99,14 +99,14 @@ public class ConfigFile implements Serializable {
 
   /**
    * The absolute path that this configuration file should be mounted as, in the
-   * application container.
+   * service container.
    **/
   public ConfigFile destFile(String destFile) {
     this.destFile = destFile;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "The absolute path that this configuration file should be mounted as, in the application container.")
+  @ApiModelProperty(example = "null", value = "The absolute path that this configuration file should be mounted as, in the service container.")
   @JsonProperty("dest_file")
   public String getDestFile() {
     return destFile;

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ConfigFormat.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ConfigFormat.java


+ 8 - 8
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Configuration.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Configuration.java

@@ -34,14 +34,14 @@ import java.util.Map;
 import java.util.Objects;
 
 /**
- * Set of configuration properties that can be injected into the application
+ * Set of configuration properties that can be injected into the service
  * components via envs, files and custom pluggable helper docker containers.
  * Files of several standard formats like xml, properties, json, yaml and
  * templates will be supported.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "Set of configuration properties that can be injected into the application components via envs, files and custom pluggable helper docker containers. Files of several standard formats like xml, properties, json, yaml and templates will be supported.")
+@ApiModel(description = "Set of configuration properties that can be injected into the service components via envs, files and custom pluggable helper docker containers. Files of several standard formats like xml, properties, json, yaml and templates will be supported.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 @JsonInclude(JsonInclude.Include.NON_NULL)
 public class Configuration implements Serializable {
@@ -52,14 +52,14 @@ public class Configuration implements Serializable {
   private List<ConfigFile> files = new ArrayList<ConfigFile>();
 
   /**
-   * A blob of key-value pairs of common application properties.
+   * A blob of key-value pairs of common service properties.
    **/
   public Configuration properties(Map<String, String> properties) {
     this.properties = properties;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "A blob of key-value pairs of common application properties.")
+  @ApiModelProperty(example = "null", value = "A blob of key-value pairs of common service properties.")
   @JsonProperty("properties")
   public Map<String, String> getProperties() {
     return properties;
@@ -71,7 +71,7 @@ public class Configuration implements Serializable {
 
   /**
    * A blob of key-value pairs which will be appended to the default system
-   * properties and handed off to the application at start time. All placeholder
+   * properties and handed off to the service at start time. All placeholder
    * references to properties will be substituted before injection.
    **/
   public Configuration env(Map<String, String> env) {
@@ -79,7 +79,7 @@ public class Configuration implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "A blob of key-value pairs which will be appended to the default system properties and handed off to the application at start time. All placeholder references to properties will be substituted before injection.")
+  @ApiModelProperty(example = "null", value = "A blob of key-value pairs which will be appended to the default system properties and handed off to the service at start time. All placeholder references to properties will be substituted before injection.")
   @JsonProperty("env")
   public Map<String, String> getEnv() {
     return env;
@@ -91,14 +91,14 @@ public class Configuration implements Serializable {
 
   /**
    * Array of list of files that needs to be created and made available as
-   * volumes in the application component containers.
+   * volumes in the service component containers.
    **/
   public Configuration files(List<ConfigFile> files) {
     this.files = files;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Array of list of files that needs to be created and made available as volumes in the application component containers.")
+  @ApiModelProperty(example = "null", value = "Array of list of files that needs to be created and made available as volumes in the service component containers.")
   @JsonProperty("files")
   public List<ConfigFile> getFiles() {
     return files;

+ 6 - 6
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Container.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Container.java

@@ -32,11 +32,11 @@ import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 
 /**
- * An instance of a running application container.
+ * An instance of a running service container.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "An instance of a running application container")
+@ApiModel(description = "An instance of a running service container")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 @XmlRootElement
 @JsonInclude(JsonInclude.Include.NON_NULL)
@@ -55,7 +55,7 @@ public class Container extends BaseResource {
   private Boolean privilegedContainer = null;
 
   /**
-   * Unique container id of a running application, e.g.
+   * Unique container id of a running service, e.g.
    * container_e3751_1458061340047_0008_01_000002.
    **/
   public Container id(String id) {
@@ -63,7 +63,7 @@ public class Container extends BaseResource {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Unique container id of a running application, e.g. container_e3751_1458061340047_0008_01_000002.")
+  @ApiModelProperty(example = "null", value = "Unique container id of a running service, e.g. container_e3751_1458061340047_0008_01_000002.")
   @JsonProperty("id")
   public String getId() {
     return id;
@@ -155,14 +155,14 @@ public class Container extends BaseResource {
   }
 
   /**
-   * State of the container of an application.
+   * State of the container of an service.
    **/
   public Container state(ContainerState state) {
     this.state = state;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "State of the container of an application.")
+  @ApiModelProperty(example = "null", value = "State of the container of an service.")
   @JsonProperty("state")
   public ContainerState getState() {
     return state;

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ContainerState.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ContainerState.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Error.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Error.java


+ 7 - 7
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/PlacementPolicy.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/PlacementPolicy.java

@@ -28,12 +28,12 @@ import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 
 /**
- * Placement policy of an instance of an application. This feature is in the
+ * Placement policy of an instance of an service. This feature is in the
  * works in YARN-4902.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "Placement policy of an instance of an application. This feature is in the works in YARN-4902.")
+@ApiModel(description = "Placement policy of an instance of an service. This feature is in the works in YARN-4902.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 public class PlacementPolicy implements Serializable {
   private static final long serialVersionUID = 4341110649551172231L;
@@ -41,10 +41,10 @@ public class PlacementPolicy implements Serializable {
   private String label = null;
 
   /**
-   * Assigns an app to a named partition of the cluster where the application
-   * desires to run (optional). If not specified all apps are submitted to a
-   * default label of the app owner. One or more labels can be setup for each
-   * application owner account with required constraints like no-preemption,
+   * Assigns a service to a named partition of the cluster where the service
+   * desires to run (optional). If not specified all services are submitted to
+   * a default label of the service owner. One or more labels can be setup for
+   * each service owner account with required constraints like no-preemption,
    * sla-99999, preemption-ok, etc.
    **/
   public PlacementPolicy label(String label) {
@@ -52,7 +52,7 @@ public class PlacementPolicy implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Assigns an app to a named partition of the cluster where the application desires to run (optional). If not specified all apps are submitted to a default label of the app owner. One or more labels can be setup for each application owner account with required constraints like no-preemption, sla-99999, preemption-ok, etc.")
+  @ApiModelProperty(example = "null", value = "Assigns a service to a named partition of the cluster where the service desires to run (optional). If not specified all services are submitted to a default label of the service owner. One or more labels can be setup for each service owner account with required constraints like no-preemption, sla-99999, preemption-ok, etc.")
   @JsonProperty("label")
   public String getLabel() {
     return label;

+ 5 - 5
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ReadinessCheck.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ReadinessCheck.java

@@ -32,13 +32,13 @@ import org.apache.hadoop.classification.InterfaceStability;
 
 /**
  * A custom command or a pluggable helper container to determine the readiness
- * of a container of a component. Readiness for every application is different.
+ * of a container of a component. Readiness for every service is different.
  * Hence the need for a simple interface, with scope to support advanced
  * usecases.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "A custom command or a pluggable helper container to determine the readiness of a container of a component. Readiness for every application is different. Hence the need for a simple interface, with scope to support advanced usecases.")
+@ApiModel(description = "A custom command or a pluggable helper container to determine the readiness of a container of a component. Readiness for every service is different. Hence the need for a simple interface, with scope to support advanced usecases.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 public class ReadinessCheck implements Serializable {
   private static final long serialVersionUID = -3836839816887186801L;
@@ -111,8 +111,8 @@ public class ReadinessCheck implements Serializable {
    * specified, this helper container typically hosts the http uri and
    * encapsulates the complex scripts required to perform actual container
    * readiness check. At the end it is expected to respond a 204 No content just
-   * like the simplified use case. This pluggable framework benefits application
-   * owners who can run applications without any packaging modifications. Note,
+   * like the simplified use case. This pluggable framework benefits service
+   * owners who can run services without any packaging modifications. Note,
    * artifacts of type docker only is supported for now.
    **/
   public ReadinessCheck artifact(Artifact artifact) {
@@ -120,7 +120,7 @@ public class ReadinessCheck implements Serializable {
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Artifact of the pluggable readiness check helper container (optional). If specified, this helper container typically hosts the http uri and encapsulates the complex scripts required to perform actual container readiness check. At the end it is expected to respond a 204 No content just like the simplified use case. This pluggable framework benefits application owners who can run applications without any packaging modifications. Note, artifacts of type docker only is supported for now.")
+  @ApiModelProperty(example = "null", value = "Artifact of the pluggable readiness check helper container (optional). If specified, this helper container typically hosts the http uri and encapsulates the complex scripts required to perform actual container readiness check. At the end it is expected to respond a 204 No content just like the simplified use case. This pluggable framework benefits service owners who can run services without any packaging modifications. Note, artifacts of type docker only is supported for now.")
   @JsonProperty("artifact")
   public Artifact getArtifact() {
     return artifact;

+ 5 - 5
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Resource.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Resource.java

@@ -29,14 +29,14 @@ import org.apache.hadoop.classification.InterfaceStability;
 /**
  * Resource determines the amount of resources (vcores, memory, network, etc.)
  * usable by a container. This field determines the resource to be applied for
- * all the containers of a component or application. The resource specified at
- * the app (or global) level can be overriden at the component level. Only one
- * of profile OR cpu &amp; memory are exepected. It raises a validation
+ * all the containers of a component or service. The resource specified at
+ * the service (or global) level can be overriden at the component level. Only one
+ * of profile OR cpu &amp; memory are expected. It raises a validation
  * exception otherwise.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "Resource determines the amount of resources (vcores, memory, network, etc.) usable by a container. This field determines the resource to be applied for all the containers of a component or application. The resource specified at the app (or global) level can be overriden at the component level. Only one of profile OR cpu & memory are exepected. It raises a validation exception otherwise.")
+@ApiModel(description = "Resource determines the amount of resources (vcores, memory, network, etc.) usable by a container. This field determines the resource to be applied for all the containers of a component or service. The resource specified at the service (or global) level can be overriden at the component level. Only one of profile OR cpu & memory are expected. It raises a validation exception otherwise.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 public class Resource extends BaseResource implements Cloneable {
   private static final long serialVersionUID = -6431667797380250037L;
@@ -107,7 +107,7 @@ public class Resource extends BaseResource implements Cloneable {
     if (this.memory == null) {
       return 0;
     }
-    return Long.valueOf(memory);
+    return Long.parseLong(memory);
   }
 
   @Override

+ 66 - 66
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Application.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/Service.java

@@ -36,17 +36,17 @@ import java.util.Map;
 import java.util.Objects;
 
 /**
- * An Application resource has the following attributes.
+ * An Service resource has the following attributes.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "An Application resource has the following attributes.")
+@ApiModel(description = "An Service resource has the following attributes.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 @XmlRootElement
 @JsonInclude(JsonInclude.Include.NON_NULL)
 @JsonPropertyOrder({ "name", "state", "resource", "number_of_containers",
     "lifetime", "containers" })
-public class Application extends BaseResource {
+public class Service extends BaseResource {
   private static final long serialVersionUID = -4491694636566094885L;
 
   private String name = null;
@@ -62,19 +62,19 @@ public class Application extends BaseResource {
   private List<Component> components = new ArrayList<>();
   private Configuration configuration = new Configuration();
   private List<Container> containers = new ArrayList<>();
-  private ApplicationState state = null;
+  private ServiceState state = null;
   private Map<String, String> quicklinks = new HashMap<>();
   private String queue = null;
 
   /**
-   * A unique application name.
+   * A unique service name.
    **/
-  public Application name(String name) {
+  public Service name(String name) {
     this.name = name;
     return this;
   }
 
-  @ApiModelProperty(example = "null", required = true, value = "A unique application name.")
+  @ApiModelProperty(example = "null", required = true, value = "A unique service name.")
   @JsonProperty("name")
   public String getName() {
     return name;
@@ -85,14 +85,14 @@ public class Application extends BaseResource {
   }
 
   /**
-   * A unique application id.
+   * A unique service id.
    **/
-  public Application id(String id) {
+  public Service id(String id) {
     this.id = id;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "A unique application id.")
+  @ApiModelProperty(example = "null", value = "A unique service id.")
   @JsonProperty("id")
   public String getId() {
     return id;
@@ -103,15 +103,15 @@ public class Application extends BaseResource {
   }
 
   /**
-   * Artifact of single-component applications. Mandatory if components
+   * Artifact of single-component services. Mandatory if components
    * attribute is not specified.
    **/
-  public Application artifact(Artifact artifact) {
+  public Service artifact(Artifact artifact) {
     this.artifact = artifact;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Artifact of single-component applications. Mandatory if components attribute is not specified.")
+  @ApiModelProperty(example = "null", value = "Artifact of single-component services. Mandatory if components attribute is not specified.")
   @JsonProperty("artifact")
   public Artifact getArtifact() {
     return artifact;
@@ -122,17 +122,17 @@ public class Application extends BaseResource {
   }
 
   /**
-   * Resource of single-component applications or the global default for
-   * multi-component applications. Mandatory if it is a single-component
-   * application and if cpus and memory are not specified at the Application
+   * Resource of single-component services or the global default for
+   * multi-component services. Mandatory if it is a single-component
+   * service and if cpus and memory are not specified at the Service
    * level.
    **/
-  public Application resource(Resource resource) {
+  public Service resource(Resource resource) {
     this.resource = resource;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Resource of single-component applications or the global default for multi-component applications. Mandatory if it is a single-component application and if cpus and memory are not specified at the Application level.")
+  @ApiModelProperty(example = "null", value = "Resource of single-component services or the global default for multi-component services. Mandatory if it is a single-component service and if cpus and memory are not specified at the Service level.")
   @JsonProperty("resource")
   public Resource getResource() {
     return resource;
@@ -143,18 +143,18 @@ public class Application extends BaseResource {
   }
 
   /**
-   * The custom launch command of an application component (optional). If not
-   * specified for applications with docker images say, it will default to the
+   * The custom launch command of an service component (optional). If not
+   * specified for services with docker images say, it will default to the
    * default start command of the image. If there is a single component in this
-   * application, you can specify this without the need to have a 'components'
+   * service, you can specify this without the need to have a 'components'
    * section.
    **/
-  public Application launchCommand(String launchCommand) {
+  public Service launchCommand(String launchCommand) {
     this.launchCommand = launchCommand;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "The custom launch command of an application component (optional). If not specified for applications with docker images say, it will default to the default start command of the image. If there is a single component in this application, you can specify this without the need to have a 'components' section.")
+  @ApiModelProperty(example = "null", value = "The custom launch command of an service component (optional). If not specified for services with docker images say, it will default to the default start command of the image. If there is a single component in this service, you can specify this without the need to have a 'components' section.")
   @JsonProperty("launch_command")
   public String getLaunchCommand() {
     return launchCommand;
@@ -166,14 +166,14 @@ public class Application extends BaseResource {
   }
 
   /**
-   * The time when the application was created, e.g. 2016-03-16T01:01:49.000Z.
+   * The time when the service was created, e.g. 2016-03-16T01:01:49.000Z.
    **/
-  public Application launchTime(Date launchTime) {
+  public Service launchTime(Date launchTime) {
     this.launchTime = launchTime == null ? null : (Date) launchTime.clone();
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "The time when the application was created, e.g. 2016-03-16T01:01:49.000Z.")
+  @ApiModelProperty(example = "null", value = "The time when the service was created, e.g. 2016-03-16T01:01:49.000Z.")
   @JsonProperty("launch_time")
   public Date getLaunchTime() {
     return launchTime == null ? null : (Date) launchTime.clone();
@@ -185,15 +185,15 @@ public class Application extends BaseResource {
   }
 
   /**
-   * Number of containers for each app-component in the application. Each
-   * app-component can further override this app-level global default.
+   * Number of containers for each component in the service. Each
+   * component can further override this service-level global default.
    **/
-  public Application numberOfContainers(Long numberOfContainers) {
+  public Service numberOfContainers(Long numberOfContainers) {
     this.numberOfContainers = numberOfContainers;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Number of containers for each app-component in the application. Each app-component can further override this app-level global default.")
+  @ApiModelProperty(example = "null", value = "Number of containers for each component in the service. Each component can further override this service-level global default.")
   @JsonProperty("number_of_containers")
   public Long getNumberOfContainers() {
     return numberOfContainers;
@@ -206,17 +206,17 @@ public class Application extends BaseResource {
 
   /**
    * In get response this provides the total number of running containers for
-   * this application (across all components) at the time of request. Note, a
+   * this service (across all components) at the time of request. Note, a
    * subsequent request can return a different number as and when more
    * containers get allocated until it reaches the total number of containers or
    * if a flex request has been made between the two requests.
    **/
-  public Application numberOfRunningContainers(Long numberOfRunningContainers) {
+  public Service numberOfRunningContainers(Long numberOfRunningContainers) {
     this.numberOfRunningContainers = numberOfRunningContainers;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "In get response this provides the total number of running containers for this application (across all components) at the time of request. Note, a subsequent request can return a different number as and when more containers get allocated until it reaches the total number of containers or if a flex request has been made between the two requests.")
+  @ApiModelProperty(example = "null", value = "In get response this provides the total number of running containers for this service (across all components) at the time of request. Note, a subsequent request can return a different number as and when more containers get allocated until it reaches the total number of containers or if a flex request has been made between the two requests.")
   @JsonProperty("number_of_running_containers")
   public Long getNumberOfRunningContainers() {
     return numberOfRunningContainers;
@@ -228,16 +228,16 @@ public class Application extends BaseResource {
   }
 
   /**
-   * Life time (in seconds) of the application from the time it reaches the
+   * Life time (in seconds) of the service from the time it reaches the
    * RUNNING_BUT_UNREADY state (after which it is automatically destroyed by YARN). For
    * unlimited lifetime do not set a lifetime value.
    **/
-  public Application lifetime(Long lifetime) {
+  public Service lifetime(Long lifetime) {
     this.lifetime = lifetime;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Life time (in seconds) of the application from the time it reaches the RUNNING_BUT_UNREADY state (after which it is automatically destroyed by YARN). For unlimited lifetime do not set a lifetime value.")
+  @ApiModelProperty(example = "null", value = "Life time (in seconds) of the service from the time it reaches the RUNNING_BUT_UNREADY state (after which it is automatically destroyed by YARN). For unlimited lifetime do not set a lifetime value.")
   @JsonProperty("lifetime")
   public Long getLifetime() {
     return lifetime;
@@ -249,18 +249,18 @@ public class Application extends BaseResource {
 
   /**
    * Advanced scheduling and placement policies (optional). If not specified, it
-   * defaults to the default placement policy of the app owner. The design of
+   * defaults to the default placement policy of the service owner. The design of
    * placement policies are in the works. It is not very clear at this point,
-   * how policies in conjunction with labels be exposed to application owners.
+   * how policies in conjunction with labels be exposed to service owners.
    * This is a placeholder for now. The advanced structure of this attribute
    * will be determined by YARN-4902.
    **/
-  public Application placementPolicy(PlacementPolicy placementPolicy) {
+  public Service placementPolicy(PlacementPolicy placementPolicy) {
     this.placementPolicy = placementPolicy;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Advanced scheduling and placement policies (optional). If not specified, it defaults to the default placement policy of the app owner. The design of placement policies are in the works. It is not very clear at this point, how policies in conjunction with labels be exposed to application owners. This is a placeholder for now. The advanced structure of this attribute will be determined by YARN-4902.")
+  @ApiModelProperty(example = "null", value = "Advanced scheduling and placement policies (optional). If not specified, it defaults to the default placement policy of the service owner. The design of placement policies are in the works. It is not very clear at this point, how policies in conjunction with labels be exposed to service owners. This is a placeholder for now. The advanced structure of this attribute will be determined by YARN-4902.")
   @JsonProperty("placement_policy")
   public PlacementPolicy getPlacementPolicy() {
     return placementPolicy;
@@ -272,14 +272,14 @@ public class Application extends BaseResource {
   }
 
   /**
-   * Components of an application.
+   * Components of an service.
    **/
-  public Application components(List<Component> components) {
+  public Service components(List<Component> components) {
     this.components = components;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Components of an application.")
+  @ApiModelProperty(example = "null", value = "Components of an service.")
   @JsonProperty("components")
   public List<Component> getComponents() {
     return components;
@@ -303,16 +303,16 @@ public class Application extends BaseResource {
   }
 
   /**
-   * Config properties of an application. Configurations provided at the
-   * application/global level are available to all the components. Specific
+   * Config properties of an service. Configurations provided at the
+   * service/global level are available to all the components. Specific
    * properties can be overridden at the component level.
    **/
-  public Application configuration(Configuration configuration) {
+  public Service configuration(Configuration configuration) {
     this.configuration = configuration;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Config properties of an application. Configurations provided at the application/global level are available to all the components. Specific properties can be overridden at the component level.")
+  @ApiModelProperty(example = "null", value = "Config properties of an service. Configurations provided at the service/global level are available to all the components. Specific properties can be overridden at the component level.")
   @JsonProperty("configuration")
   public Configuration getConfiguration() {
     return configuration;
@@ -323,16 +323,16 @@ public class Application extends BaseResource {
   }
 
   /**
-   * Containers of a started application. Specifying a value for this attribute
+   * Containers of a started service. Specifying a value for this attribute
    * for the POST payload raises a validation error. This blob is available only
-   * in the GET response of a started application.
+   * in the GET response of a started service.
    **/
-  public Application containers(List<Container> containers) {
+  public Service containers(List<Container> containers) {
     this.containers = containers;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Containers of a started application. Specifying a value for this attribute for the POST payload raises a validation error. This blob is available only in the GET response of a started application.")
+  @ApiModelProperty(example = "null", value = "Containers of a started service. Specifying a value for this attribute for the POST payload raises a validation error. This blob is available only in the GET response of a started service.")
   @JsonProperty("containers")
   public List<Container> getContainers() {
     return containers;
@@ -347,34 +347,34 @@ public class Application extends BaseResource {
   }
 
   /**
-   * State of the application. Specifying a value for this attribute for the
+   * State of the service. Specifying a value for this attribute for the
    * POST payload raises a validation error. This attribute is available only in
-   * the GET response of a started application.
+   * the GET response of a started service.
    **/
-  public Application state(ApplicationState state) {
+  public Service state(ServiceState state) {
     this.state = state;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "State of the application. Specifying a value for this attribute for the POST payload raises a validation error. This attribute is available only in the GET response of a started application.")
+  @ApiModelProperty(example = "null", value = "State of the service. Specifying a value for this attribute for the POST payload raises a validation error. This attribute is available only in the GET response of a started service.")
   @JsonProperty("state")
-  public ApplicationState getState() {
+  public ServiceState getState() {
     return state;
   }
 
-  public void setState(ApplicationState state) {
+  public void setState(ServiceState state) {
     this.state = state;
   }
 
   /**
-   * A blob of key-value pairs of quicklinks to be exported for an application.
+   * A blob of key-value pairs of quicklinks to be exported for an service.
    **/
-  public Application quicklinks(Map<String, String> quicklinks) {
+  public Service quicklinks(Map<String, String> quicklinks) {
     this.quicklinks = quicklinks;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "A blob of key-value pairs of quicklinks to be exported for an application.")
+  @ApiModelProperty(example = "null", value = "A blob of key-value pairs of quicklinks to be exported for an service.")
   @JsonProperty("quicklinks")
   public Map<String, String> getQuicklinks() {
     return quicklinks;
@@ -385,14 +385,14 @@ public class Application extends BaseResource {
   }
 
   /**
-   * The YARN queue that this application should be submitted to.
+   * The YARN queue that this service should be submitted to.
    **/
-  public Application queue(String queue) {
+  public Service queue(String queue) {
     this.queue = queue;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "The YARN queue that this application should be submitted to.")
+  @ApiModelProperty(example = "null", value = "The YARN queue that this service should be submitted to.")
   @JsonProperty("queue")
   public String getQueue() {
     return queue;
@@ -410,8 +410,8 @@ public class Application extends BaseResource {
     if (o == null || getClass() != o.getClass()) {
       return false;
     }
-    Application application = (Application) o;
-    return Objects.equals(this.name, application.name);
+    Service service = (Service) o;
+    return Objects.equals(this.name, service.name);
   }
 
   @Override
@@ -422,7 +422,7 @@ public class Application extends BaseResource {
   @Override
   public String toString() {
     StringBuilder sb = new StringBuilder();
-    sb.append("class Application {\n");
+    sb.append("class Service {\n");
 
     sb.append("    name: ").append(toIndentedString(name)).append("\n");
     sb.append("    id: ").append(toIndentedString(id)).append("\n");

+ 3 - 3
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ApplicationState.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ServiceState.java

@@ -22,12 +22,12 @@ import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 
 /**
- * The current state of an application.
+ * The current state of an service.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "The current state of an application.")
+@ApiModel(description = "The current state of an service.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
-public enum ApplicationState {
+public enum ServiceState {
   ACCEPTED, STARTED, READY, STOPPED, FAILED;
 }

+ 20 - 20
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ApplicationStatus.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/api/records/ServiceStatus.java

@@ -30,33 +30,33 @@ import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
 
 /**
- * The current status of a submitted application, returned as a response to the
+ * The current status of a submitted service, returned as a response to the
  * GET API.
  **/
 @InterfaceAudience.Public
 @InterfaceStability.Unstable
-@ApiModel(description = "The current status of a submitted application, returned as a response to the GET API.")
+@ApiModel(description = "The current status of a submitted service, returned as a response to the GET API.")
 @javax.annotation.Generated(value = "class io.swagger.codegen.languages.JavaClientCodegen", date = "2016-06-02T08:15:05.615-07:00")
 @XmlRootElement
 @JsonInclude(JsonInclude.Include.NON_NULL)
-public class ApplicationStatus extends BaseResource {
+public class ServiceStatus extends BaseResource {
   private static final long serialVersionUID = -3469885905347851034L;
 
   private String diagnostics = null;
-  private ApplicationState state = null;
+  private ServiceState state = null;
   private Integer code = null;
 
   /**
    * Diagnostic information (if any) for the reason of the current state of the
-   * application. It typically has a non-null value, if the application is in a
+   * service. It typically has a non-null value, if the service is in a
    * non-running state.
    **/
-  public ApplicationStatus diagnostics(String diagnostics) {
+  public ServiceStatus diagnostics(String diagnostics) {
     this.diagnostics = diagnostics;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Diagnostic information (if any) for the reason of the current state of the application. It typically has a non-null value, if the application is in a non-running state.")
+  @ApiModelProperty(example = "null", value = "Diagnostic information (if any) for the reason of the current state of the service. It typically has a non-null value, if the service is in a non-running state.")
   @JsonProperty("diagnostics")
   public String getDiagnostics() {
     return diagnostics;
@@ -67,33 +67,33 @@ public class ApplicationStatus extends BaseResource {
   }
 
   /**
-   * Application state.
+   * Service state.
    **/
-  public ApplicationStatus state(ApplicationState state) {
+  public ServiceStatus state(ServiceState state) {
     this.state = state;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "Application state.")
+  @ApiModelProperty(example = "null", value = "Service state.")
   @JsonProperty("state")
-  public ApplicationState getState() {
+  public ServiceState getState() {
     return state;
   }
 
-  public void setState(ApplicationState state) {
+  public void setState(ServiceState state) {
     this.state = state;
   }
 
   /**
-   * An error code specific to a scenario which app owners should be able to use
+   * An error code specific to a scenario which service owners should be able to use
    * to understand the failure in addition to the diagnostic information.
    **/
-  public ApplicationStatus code(Integer code) {
+  public ServiceStatus code(Integer code) {
     this.code = code;
     return this;
   }
 
-  @ApiModelProperty(example = "null", value = "An error code specific to a scenario which app owners should be able to use to understand the failure in addition to the diagnostic information.")
+  @ApiModelProperty(example = "null", value = "An error code specific to a scenario which service owners should be able to use to understand the failure in addition to the diagnostic information.")
   @JsonProperty("code")
   public Integer getCode() {
     return code;
@@ -111,10 +111,10 @@ public class ApplicationStatus extends BaseResource {
     if (o == null || getClass() != o.getClass()) {
       return false;
     }
-    ApplicationStatus applicationStatus = (ApplicationStatus) o;
-    return Objects.equals(this.diagnostics, applicationStatus.diagnostics)
-        && Objects.equals(this.state, applicationStatus.state)
-        && Objects.equals(this.code, applicationStatus.code);
+    ServiceStatus serviceStatus = (ServiceStatus) o;
+    return Objects.equals(this.diagnostics, serviceStatus.diagnostics)
+        && Objects.equals(this.state, serviceStatus.state)
+        && Objects.equals(this.code, serviceStatus.code);
   }
 
   @Override
@@ -125,7 +125,7 @@ public class ApplicationStatus extends BaseResource {
   @Override
   public String toString() {
     StringBuilder sb = new StringBuilder();
-    sb.append("class ApplicationStatus {\n");
+    sb.append("class ServiceStatus {\n");
 
     sb.append("    diagnostics: ").append(toIndentedString(diagnostics))
         .append("\n");

+ 13 - 5
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/ClientAMProxy.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/ClientAMProxy.java

@@ -28,21 +28,29 @@ import org.apache.hadoop.yarn.service.conf.YarnServiceConf;
 
 import java.net.InetSocketAddress;
 
+import static org.apache.hadoop.io.retry.RetryPolicies.TRY_ONCE_THEN_FAIL;
+
 public class ClientAMProxy extends ServerProxy{
 
   public static <T> T createProxy(final Configuration conf,
       final Class<T> protocol, final UserGroupInformation ugi,
       final YarnRPC rpc, final InetSocketAddress serverAddress) {
-
-    RetryPolicy retryPolicy =
-        createRetryPolicy(conf, YarnServiceConf.CLIENT_AM_RETRY_MAX_WAIT_MS,
-            15 * 60 * 1000, YarnServiceConf.CLIENT_AM_RETRY_MAX_INTERVAL_MS,
-            2 * 1000);
     Configuration confClone = new Configuration(conf);
     confClone.setInt(
         CommonConfigurationKeysPublic.IPC_CLIENT_CONNECT_MAX_RETRIES_KEY, 0);
     confClone.setInt(CommonConfigurationKeysPublic.
         IPC_CLIENT_CONNECT_MAX_RETRIES_ON_SOCKET_TIMEOUTS_KEY, 0);
+    RetryPolicy retryPolicy;
+
+    if (conf.getLong(YarnServiceConf.CLIENT_AM_RETRY_MAX_WAIT_MS, 0) == 0) {
+      // by default no retry
+      retryPolicy = TRY_ONCE_THEN_FAIL;
+    } else {
+      retryPolicy =
+          createRetryPolicy(conf, YarnServiceConf.CLIENT_AM_RETRY_MAX_WAIT_MS,
+              15 * 60 * 1000, YarnServiceConf.CLIENT_AM_RETRY_MAX_INTERVAL_MS,
+              2 * 1000);
+    }
     return createRetriableProxy(confClone, protocol, ugi, rpc, serverAddress,
         retryPolicy);
   }

+ 11 - 3
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/ServiceCLI.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/ServiceCLI.java

@@ -20,12 +20,14 @@ package org.apache.hadoop.yarn.service.client;
 
 import org.apache.commons.lang.StringUtils;
 import org.apache.hadoop.yarn.conf.YarnConfiguration;
-import org.apache.hadoop.yarn.service.api.records.Application;
+import org.apache.hadoop.yarn.service.api.records.Service;
 import org.apache.hadoop.yarn.service.client.params.ClientArgs;
 import org.apache.hadoop.yarn.service.exceptions.BadCommandArgumentsException;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
+import java.io.FileNotFoundException;
+
 import static org.apache.hadoop.yarn.service.client.params.SliderActions.*;
 
 public class ServiceCLI {
@@ -49,11 +51,17 @@ public class ServiceCLI {
       client.actionCreate(args.getActionCreateArgs());
       break;
     case ACTION_STATUS:
-      Application app = client.getStatus(args.getClusterName());
+      Service app = client.getStatus(args.getClusterName());
       System.out.println(app);
       break;
     case ACTION_FLEX:
-      client.actionFlexByCLI(args);
+      try {
+        client.actionFlexByCLI(args);
+      } catch (FileNotFoundException e) {
+        System.err.println(
+            args.getClusterName() + " doesn't exist: " + e.getMessage());
+        return -1;
+      }
       break;
     case ACTION_STOP:
       client.actionStop(args.getClusterName(), false);

+ 199 - 179
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/ServiceClient.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/ServiceClient.java

@@ -45,6 +45,7 @@ import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
 import org.apache.hadoop.yarn.api.records.ApplicationTimeout;
 import org.apache.hadoop.yarn.api.records.ApplicationTimeoutType;
 import org.apache.hadoop.yarn.api.records.ContainerLaunchContext;
+import org.apache.hadoop.yarn.api.records.FinalApplicationStatus;
 import org.apache.hadoop.yarn.api.records.LocalResource;
 import org.apache.hadoop.yarn.api.records.LocalResourceType;
 import org.apache.hadoop.yarn.api.records.Resource;
@@ -61,8 +62,9 @@ import org.apache.hadoop.yarn.proto.ClientAMProtocol.GetStatusResponseProto;
 import org.apache.hadoop.yarn.proto.ClientAMProtocol.StopRequestProto;
 import org.apache.hadoop.yarn.service.ClientAMProtocol;
 import org.apache.hadoop.yarn.service.ServiceMaster;
-import org.apache.hadoop.yarn.service.api.records.Application;
+import org.apache.hadoop.yarn.service.api.records.Service;
 import org.apache.hadoop.yarn.service.api.records.Component;
+import org.apache.hadoop.yarn.service.api.records.ServiceState;
 import org.apache.hadoop.yarn.service.client.params.AbstractClusterBuildingActionArgs;
 import org.apache.hadoop.yarn.service.client.params.ActionDependencyArgs;
 import org.apache.hadoop.yarn.service.client.params.ActionFlexArgs;
@@ -107,6 +109,7 @@ import static org.apache.hadoop.yarn.api.records.YarnApplicationState.*;
 import static org.apache.hadoop.yarn.service.client.params.SliderActions.ACTION_CREATE;
 import static org.apache.hadoop.yarn.service.client.params.SliderActions.ACTION_FLEX;
 import static org.apache.hadoop.yarn.service.conf.YarnServiceConf.YARN_QUEUE;
+import static org.apache.hadoop.yarn.service.utils.ServiceApiUtil.jsonSerDeser;
 import static org.apache.hadoop.yarn.service.utils.SliderUtils.*;
 
 @InterfaceAudience.Public
@@ -116,10 +119,10 @@ public class ServiceClient extends CompositeService
   private static final Logger LOG =
       LoggerFactory.getLogger(ServiceClient.class);
   private SliderFileSystem fs;
-  private YarnClient yarnClient;
+  //TODO disable retry so that client / rest API doesn't block?
+  protected YarnClient yarnClient;
   // Avoid looking up applicationId from fs all the time.
   private Map<String, ApplicationId> cachedAppIds = new ConcurrentHashMap<>();
-  private Map<String, ClientAMProtocol> cachedAMProxies = new ConcurrentHashMap<>();
 
   private RegistryOperations registryClient;
   private CuratorFramework curatorClient;
@@ -128,7 +131,9 @@ public class ServiceClient extends CompositeService
   private static EnumSet<YarnApplicationState> terminatedStates =
       EnumSet.of(FINISHED, FAILED, KILLED);
   private static EnumSet<YarnApplicationState> liveStates =
-      EnumSet.of(NEW, NEW_SAVING, SUBMITTED, RUNNING);
+      EnumSet.of(NEW, NEW_SAVING, SUBMITTED, ACCEPTED, RUNNING);
+  private static EnumSet<YarnApplicationState> preRunningStates =
+      EnumSet.of(NEW, NEW_SAVING, SUBMITTED, ACCEPTED);
 
   public ServiceClient() {
     super(ServiceClient.class.getName());
@@ -151,18 +156,18 @@ public class ServiceClient extends CompositeService
     super.serviceStop();
   }
 
-  private Application loadAppJsonFromLocalFS(
+  private Service loadAppJsonFromLocalFS(
       AbstractClusterBuildingActionArgs args) throws IOException {
     File file = args.getAppDef();
     Path filePath = new Path(file.getAbsolutePath());
     LOG.info("Loading app json from: " + filePath);
-    Application application = ServiceApiUtil.jsonSerDeser
+    Service service = jsonSerDeser
         .load(FileSystem.getLocal(getConfig()), filePath);
     if (args.lifetime > 0) {
-      application.setLifetime(args.lifetime);
+      service.setLifetime(args.lifetime);
     }
-    application.setName(args.getClusterName());
-    return application;
+    service.setName(args.getClusterName());
+    return service;
   }
 
   public int actionBuild(AbstractClusterBuildingActionArgs args)
@@ -170,11 +175,11 @@ public class ServiceClient extends CompositeService
     return actionBuild(loadAppJsonFromLocalFS(args));
   }
 
-  public int actionBuild(Application application)
+  public int actionBuild(Service service)
       throws YarnException, IOException {
-    Path appDir = checkAppNotExistOnHdfs(application);
-    ServiceApiUtil.validateAndResolveApplication(application, fs, getConfig());
-    createDirAndPersistApp(appDir, application);
+    Path appDir = checkAppNotExistOnHdfs(service);
+    ServiceApiUtil.validateAndResolveService(service, fs, getConfig());
+    createDirAndPersistApp(appDir, service);
     return EXIT_SUCCESS;
   }
 
@@ -184,21 +189,21 @@ public class ServiceClient extends CompositeService
     return EXIT_SUCCESS;
   }
 
-  public ApplicationId actionCreate(Application application)
+  public ApplicationId actionCreate(Service service)
       throws IOException, YarnException {
-    String appName = application.getName();
-    validateClusterName(appName);
-    ServiceApiUtil.validateAndResolveApplication(application, fs, getConfig());
-    verifyNoLiveAppInRM(appName, "create");
-    Path appDir = checkAppNotExistOnHdfs(application);
+    String serviceName = service.getName();
+    ServiceApiUtil.validateNameFormat(serviceName, getConfig());
+    ServiceApiUtil.validateAndResolveService(service, fs, getConfig());
+    verifyNoLiveAppInRM(serviceName, "create");
+    Path appDir = checkAppNotExistOnHdfs(service);
 
     // Write the definition first and then submit - AM will read the definition
-    createDirAndPersistApp(appDir, application);
-    ApplicationId appId = submitApp(application);
-    cachedAppIds.put(appName, appId);
-    application.setId(appId.toString());
+    createDirAndPersistApp(appDir, service);
+    ApplicationId appId = submitApp(service);
+    cachedAppIds.put(serviceName, appId);
+    service.setId(appId.toString());
     // update app definition with appId
-    persistAppDef(appDir, application);
+    persistAppDef(appDir, service);
     return appId;
   }
 
@@ -208,17 +213,21 @@ public class ServiceClient extends CompositeService
     ActionFlexArgs flexArgs = args.getActionFlexArgs();
     Map<String, Long> componentCounts =
         new HashMap<>(flexArgs.getComponentMap().size());
-    Application persistedApp =
-        ServiceApiUtil.loadApplication(fs, flexArgs.getClusterName());
-    if (!StringUtils.isEmpty(persistedApp.getId())) {
-      cachedAppIds.put(persistedApp.getName(),
-          ApplicationId.fromString(persistedApp.getId()));
+    Service persistedService =
+        ServiceApiUtil.loadService(fs, flexArgs.getClusterName());
+    if (!StringUtils.isEmpty(persistedService.getId())) {
+      cachedAppIds.put(persistedService.getName(),
+          ApplicationId.fromString(persistedService.getId()));
+    } else {
+      throw new YarnException(persistedService.getName()
+          + " appId is null, may be not submitted to YARN yet");
     }
+
     for (Map.Entry<String, String> entry : flexArgs.getComponentMap()
         .entrySet()) {
       String compName = entry.getKey();
-      ServiceApiUtil.validateCompName(compName);
-      Component component = persistedApp.getComponent(compName);
+      ServiceApiUtil.validateNameFormat(compName, getConfig());
+      Component component = persistedService.getComponent(compName);
       if (component == null) {
         throw new IllegalArgumentException(entry.getKey() + " does not exist !");
       }
@@ -230,7 +239,7 @@ public class ServiceClient extends CompositeService
     if (componentCounts.size() == 0) {
       actionHelp(ACTION_FLEX, args);
     }
-    flexComponents(args.getClusterName(), componentCounts, persistedApp);
+    flexComponents(args.getClusterName(), componentCounts, persistedService);
     return EXIT_SUCCESS;
   }
 
@@ -258,19 +267,23 @@ public class ServiceClient extends CompositeService
   }
 
   // Called by Rest Service
-  public Map<String, Long> flexByRestService(String appName,
+  public Map<String, Long> flexByRestService(String serviceName,
       Map<String, Long> componentCounts) throws YarnException, IOException {
     // load app definition
-    Application persistedApp = ServiceApiUtil.loadApplication(fs, appName);
-    cachedAppIds.put(persistedApp.getName(),
-        ApplicationId.fromString(persistedApp.getId()));
-    return flexComponents(appName, componentCounts, persistedApp);
+    Service persistedService = ServiceApiUtil.loadService(fs, serviceName);
+    if (StringUtils.isEmpty(persistedService.getId())) {
+      throw new YarnException(
+          serviceName + " appId is null, may be not submitted to YARN yet");
+    }
+    cachedAppIds.put(persistedService.getName(),
+        ApplicationId.fromString(persistedService.getId()));
+    return flexComponents(serviceName, componentCounts, persistedService);
   }
 
-  private Map<String, Long> flexComponents(String appName,
-      Map<String, Long> componentCounts, Application persistedApp)
+  private Map<String, Long> flexComponents(String serviceName,
+      Map<String, Long> componentCounts, Service persistedService)
       throws YarnException, IOException {
-    validateClusterName(appName);
+    ServiceApiUtil.validateNameFormat(serviceName, getConfig());
 
     Map<String, Long> original = new HashMap<>(componentCounts.size());
 
@@ -278,7 +291,7 @@ public class ServiceClient extends CompositeService
     FlexComponentsRequestProto.Builder requestBuilder =
         FlexComponentsRequestProto.newBuilder();
 
-    for (Component persistedComp : persistedApp.getComponents()) {
+    for (Component persistedComp : persistedService.getComponents()) {
       String name = persistedComp.getName();
       if (componentCounts.containsKey(persistedComp.getName())) {
         original.put(name, persistedComp.getNumberOfContainers());
@@ -295,15 +308,24 @@ public class ServiceClient extends CompositeService
       throw new YarnException("Components " + componentCounts.keySet()
           + " do not exist in app definition.");
     }
-    ServiceApiUtil.jsonSerDeser
-        .save(fs.getFileSystem(), ServiceApiUtil.getAppJsonPath(fs, appName),
-            persistedApp, true);
-    ClientAMProtocol proxy = getAMProxy(appName);
-    if (proxy == null) {
-      String message = appName + " is not running";
+    jsonSerDeser
+        .save(fs.getFileSystem(), ServiceApiUtil.getServiceJsonPath(fs, serviceName),
+            persistedService, true);
+
+    ApplicationReport appReport =
+        yarnClient.getApplicationReport(getAppId(serviceName));
+    if (appReport.getYarnApplicationState() != RUNNING) {
+      String message =
+          serviceName + " is at " + appReport.getYarnApplicationState()
+              + " state, flex can only be invoked when service is running";
       LOG.error(message);
       throw new YarnException(message);
     }
+    if (StringUtils.isEmpty(appReport.getHost())) {
+      throw new YarnException(serviceName + " AM hostname is empty");
+    }
+    ClientAMProtocol proxy =
+        createAMProxy(appReport.getHost(), appReport.getRpcPort());
     proxy.flexComponents(requestBuilder.build());
     for (Map.Entry<String, Long> entry : original.entrySet()) {
       LOG.info("[COMPONENT {}]: number of containers changed from {} to {}",
@@ -313,31 +335,40 @@ public class ServiceClient extends CompositeService
     return original;
   }
 
-  public int actionStop(String appName, boolean waitForAppStopped)
+  public int actionStop(String serviceName, boolean waitForAppStopped)
       throws YarnException, IOException {
-    validateClusterName(appName);
-    getAppId(appName);
-    ApplicationId currentAppId = cachedAppIds.get(appName);
+    ServiceApiUtil.validateNameFormat(serviceName, getConfig());
+    ApplicationId currentAppId = getAppId(serviceName);
     ApplicationReport report = yarnClient.getApplicationReport(currentAppId);
     if (terminatedStates.contains(report.getYarnApplicationState())) {
-      LOG.info("Application {} is already in a terminated state {}", appName,
+      LOG.info("Service {} is already in a terminated state {}", serviceName,
           report.getYarnApplicationState());
       return EXIT_SUCCESS;
     }
-    LOG.info("Stopping application {}, with appId = {}", appName, currentAppId);
+    if (preRunningStates.contains(report.getYarnApplicationState())) {
+      String msg = serviceName + " is at " + report.getYarnApplicationState()
+          + ", forcefully killed by user!";
+      yarnClient.killApplication(currentAppId, msg);
+      LOG.info(msg);
+      return EXIT_SUCCESS;
+    }
+    if (StringUtils.isEmpty(report.getHost())) {
+      throw new YarnException(serviceName + " AM hostname is empty");
+    }
+    LOG.info("Stopping service {}, with appId = {}", serviceName, currentAppId);
     try {
-      ClientAMProtocol proxy = getAMProxy(appName, report);
-      cachedAppIds.remove(appName);
-      cachedAMProxies.remove(appName);
+      ClientAMProtocol proxy =
+          createAMProxy(report.getHost(), report.getRpcPort());
+      cachedAppIds.remove(serviceName);
       if (proxy != null) {
         // try to stop the app gracefully.
         StopRequestProto request = StopRequestProto.newBuilder().build();
         proxy.stop(request);
-        LOG.info("Application " + appName + " is being gracefully stopped...");
+        LOG.info("Service " + serviceName + " is being gracefully stopped...");
       } else {
         yarnClient.killApplication(currentAppId,
-            appName + " is forcefully killed by user!");
-        LOG.info("Forcefully kill the application: " + appName);
+            serviceName + " is forcefully killed by user!");
+        LOG.info("Forcefully kill the service: " + serviceName);
         return EXIT_SUCCESS;
       }
 
@@ -351,56 +382,55 @@ public class ServiceClient extends CompositeService
         Thread.sleep(2000);
         report = yarnClient.getApplicationReport(currentAppId);
         if (terminatedStates.contains(report.getYarnApplicationState())) {
-          LOG.info("Application " + appName + " is stopped.");
+          LOG.info("Service " + serviceName + " is stopped.");
           break;
         }
         // Forcefully kill after 10 seconds.
         if ((System.currentTimeMillis() - startTime) > 10000) {
           LOG.info("Stop operation timeout stopping, forcefully kill the app "
-              + appName);
+              + serviceName);
           yarnClient.killApplication(currentAppId,
               "Forcefully kill the app by user");
           break;
         }
         if (++pollCount % 10 == 0) {
-          LOG.info("Waiting for application " + appName + " to be stopped.");
+          LOG.info("Waiting for service " + serviceName + " to be stopped.");
         }
       }
     } catch (IOException | YarnException | InterruptedException e) {
-      LOG.info("Failed to stop " + appName
+      LOG.info("Failed to stop " + serviceName
           + " gracefully, forcefully kill the app.");
       yarnClient.killApplication(currentAppId, "Forcefully kill the app");
     }
     return EXIT_SUCCESS;
   }
 
-  public int actionDestroy(String appName) throws Exception {
-    validateClusterName(appName);
-    verifyNoLiveAppInRM(appName, "Destroy");
-    Path appDir = fs.buildClusterDirPath(appName);
+  public int actionDestroy(String serviceName) throws Exception {
+    ServiceApiUtil.validateNameFormat(serviceName, getConfig());
+    verifyNoLiveAppInRM(serviceName, "Destroy");
+    Path appDir = fs.buildClusterDirPath(serviceName);
     FileSystem fileSystem = fs.getFileSystem();
     // remove from the appId cache
-    cachedAppIds.remove(appName);
-    cachedAMProxies.remove(appName);
+    cachedAppIds.remove(serviceName);
     if (fileSystem.exists(appDir)) {
       if (fileSystem.delete(appDir, true)) {
-        LOG.info("Successfully deleted application dir for " + appName + ": "
+        LOG.info("Successfully deleted service dir for " + serviceName + ": "
             + appDir);
       } else {
         String message =
-            "Failed to delete application + " + appName + " at:  " + appDir;
+            "Failed to delete service + " + serviceName + " at:  " + appDir;
         LOG.info(message);
         throw new YarnException(message);
       }
     }
-    deleteZKNode(appName);
-    String registryPath = ServiceRegistryUtils.registryPathForInstance(appName);
+    deleteZKNode(serviceName);
+    String registryPath = ServiceRegistryUtils.registryPathForInstance(serviceName);
     try {
       getRegistryClient().delete(registryPath, true);
     } catch (IOException e) {
       LOG.warn("Error deleting registry entry {}", registryPath, e);
     }
-    LOG.info("Destroyed cluster {}", appName);
+    LOG.info("Destroyed cluster {}", serviceName);
     return EXIT_SUCCESS;
   }
 
@@ -454,13 +484,13 @@ public class ServiceClient extends CompositeService
     throw new UsageException(CommonArgs.usage(args, actionName));
   }
 
-  private void verifyNoLiveAppInRM(String appname, String action)
+  private void verifyNoLiveAppInRM(String serviceName, String action)
       throws IOException, YarnException {
     Set<String> types = new HashSet<>(1);
     types.add(YarnServiceConstants.APP_TYPE);
     Set<String> tags = null;
-    if (appname != null) {
-      tags = Collections.singleton(SliderUtils.createNameTag(appname));
+    if (serviceName != null) {
+      tags = Collections.singleton(SliderUtils.createNameTag(serviceName));
     }
     GetApplicationsRequest request = GetApplicationsRequest.newInstance();
     request.setApplicationTypes(types);
@@ -469,14 +499,14 @@ public class ServiceClient extends CompositeService
     List<ApplicationReport> reports = yarnClient.getApplications(request);
     if (!reports.isEmpty()) {
       throw new YarnException(
-          "Failed to " + action + " application, as " + appname
+          "Failed to " + action + " service, as " + serviceName
               + " already exists.");
     }
   }
 
-  private ApplicationId submitApp(Application app)
+  private ApplicationId submitApp(Service app)
       throws IOException, YarnException {
-    String appName = app.getName();
+    String serviceName = app.getName();
     Configuration conf = getConfig();
     Path appRootDir = fs.buildClusterDirPath(app.getName());
 
@@ -500,11 +530,11 @@ public class ServiceClient extends CompositeService
 
     // copy local slideram-log4j.properties to hdfs and add to localResources
     boolean hasAMLog4j =
-        addAMLog4jResource(appName, conf, localResources);
+        addAMLog4jResource(serviceName, conf, localResources);
     // copy jars to hdfs and add to localResources
-    addJarResource(appName, localResources);
+    addJarResource(serviceName, localResources);
     // add keytab if in secure env
-    addKeytabResourceIfSecure(fs, localResources, conf, appName);
+    addKeytabResourceIfSecure(fs, localResources, conf, serviceName);
     if (LOG.isDebugEnabled()) {
       printLocalResources(localResources);
     }
@@ -512,7 +542,7 @@ public class ServiceClient extends CompositeService
 
     // create AM CLI
     String cmdStr =
-        buildCommandLine(appName, conf, appRootDir, hasAMLog4j);
+        buildCommandLine(serviceName, conf, appRootDir, hasAMLog4j);
     submissionContext.setResource(Resource.newInstance(YarnServiceConf
         .getLong(YarnServiceConf.AM_RESOURCE_MEM, YarnServiceConf.DEFAULT_KEY_AM_RESOURCE_MEM,
             app.getConfiguration(), conf), 1));
@@ -521,10 +551,10 @@ public class ServiceClient extends CompositeService
       queue = conf.get(YARN_QUEUE, "default");
     }
     submissionContext.setQueue(queue);
-    submissionContext.setApplicationName(appName);
+    submissionContext.setApplicationName(serviceName);
     submissionContext.setApplicationType(YarnServiceConstants.APP_TYPE);
     Set<String> appTags =
-        AbstractClientProvider.createApplicationTags(appName, null, null);
+        AbstractClientProvider.createApplicationTags(serviceName, null, null);
     if (!appTags.isEmpty()) {
       submissionContext.setApplicationTags(appTags);
     }
@@ -549,7 +579,7 @@ public class ServiceClient extends CompositeService
     LOG.debug(builder.toString());
   }
 
-  private String buildCommandLine(String appName, Configuration conf,
+  private String buildCommandLine(String serviceName, Configuration conf,
       Path appRootDir, boolean hasSliderAMLog4j) throws BadConfigException {
     JavaCommandLineBuilder CLI = new JavaCommandLineBuilder();
     CLI.forceIPv4().headless();
@@ -560,9 +590,9 @@ public class ServiceClient extends CompositeService
       CLI.sysprop(SYSPROP_LOG_DIR, ApplicationConstants.LOG_DIR_EXPANSION_VAR);
     }
     CLI.add(ServiceMaster.class.getCanonicalName());
-    CLI.add(ACTION_CREATE, appName);
+    CLI.add(ACTION_CREATE, serviceName);
     //TODO debugAM CLI.add(Arguments.ARG_DEBUG)
-    CLI.add(Arguments.ARG_CLUSTER_URI, new Path(appRootDir, appName + ".json"));
+    CLI.add(Arguments.ARG_CLUSTER_URI, new Path(appRootDir, serviceName + ".json"));
     // pass the registry binding
     CLI.addConfOptionToCLI(conf, RegistryConstants.KEY_REGISTRY_ZK_ROOT,
         RegistryConstants.DEFAULT_ZK_REGISTRY_ROOT);
@@ -599,10 +629,10 @@ public class ServiceClient extends CompositeService
     return env;
   }
 
-  protected Path addJarResource(String appName,
+  protected Path addJarResource(String serviceName,
       Map<String, LocalResource> localResources)
       throws IOException, SliderException {
-    Path libPath = fs.buildClusterDirPath(appName);
+    Path libPath = fs.buildClusterDirPath(serviceName);
     ProviderUtils
         .addProviderJar(localResources, ServiceMaster.class, SERVICE_CORE_JAR, fs,
             libPath, "lib", false);
@@ -621,7 +651,7 @@ public class ServiceClient extends CompositeService
     return libPath;
   }
 
-  private boolean addAMLog4jResource(String appName, Configuration conf,
+  private boolean addAMLog4jResource(String serviceName, Configuration conf,
       Map<String, LocalResource> localResources)
       throws IOException, BadClusterStateException {
     boolean hasAMLog4j = false;
@@ -632,7 +662,7 @@ public class ServiceClient extends CompositeService
           new File(hadoopConfDir, YarnServiceConstants.YARN_SERVICE_LOG4J_FILENAME);
       if (localFile.exists()) {
         Path localFilePath = createLocalPath(localFile);
-        Path appDirPath = fs.buildClusterDirPath(appName);
+        Path appDirPath = fs.buildClusterDirPath(serviceName);
         Path remoteConfPath =
             new Path(appDirPath, YarnServiceConstants.SUBMITTED_CONF_DIR);
         Path remoteFilePath =
@@ -649,54 +679,54 @@ public class ServiceClient extends CompositeService
     return hasAMLog4j;
   }
 
-  public int actionStart(String appName) throws YarnException, IOException {
-    validateClusterName(appName);
-    Path appDir = checkAppExistOnHdfs(appName);
-    Application application = ServiceApiUtil.loadApplication(fs, appName);
-    ServiceApiUtil.validateAndResolveApplication(application, fs, getConfig());
+  public int actionStart(String serviceName) throws YarnException, IOException {
+    ServiceApiUtil.validateNameFormat(serviceName, getConfig());
+    Path appDir = checkAppExistOnHdfs(serviceName);
+    Service service = ServiceApiUtil.loadService(fs, serviceName);
+    ServiceApiUtil.validateAndResolveService(service, fs, getConfig());
     // see if it is actually running and bail out;
-    verifyNoLiveAppInRM(appName, "thaw");
-    ApplicationId appId = submitApp(application);
-    application.setId(appId.toString());
+    verifyNoLiveAppInRM(serviceName, "thaw");
+    ApplicationId appId = submitApp(service);
+    service.setId(appId.toString());
     // write app definition on to hdfs
-    createDirAndPersistApp(appDir, application);
+    createDirAndPersistApp(appDir, service);
     return 0;
   }
 
-  private Path checkAppNotExistOnHdfs(Application application)
+  private Path checkAppNotExistOnHdfs(Service service)
       throws IOException, SliderException {
-    Path appDir = fs.buildClusterDirPath(application.getName());
+    Path appDir = fs.buildClusterDirPath(service.getName());
     fs.verifyDirectoryNonexistent(
-        new Path(appDir, application.getName() + ".json"));
+        new Path(appDir, service.getName() + ".json"));
     return appDir;
   }
 
-  private Path checkAppExistOnHdfs(String appName)
+  private Path checkAppExistOnHdfs(String serviceName)
       throws IOException, SliderException {
-    Path appDir = fs.buildClusterDirPath(appName);
-    fs.verifyPathExists(new Path(appDir, appName + ".json"));
+    Path appDir = fs.buildClusterDirPath(serviceName);
+    fs.verifyPathExists(new Path(appDir, serviceName + ".json"));
     return appDir;
   }
 
-  private void createDirAndPersistApp(Path appDir, Application application)
+  private void createDirAndPersistApp(Path appDir, Service service)
       throws IOException, SliderException {
     FsPermission appDirPermission = new FsPermission("750");
     fs.createWithPermissions(appDir, appDirPermission);
-    persistAppDef(appDir, application);
+    persistAppDef(appDir, service);
   }
 
-  private void persistAppDef(Path appDir, Application application)
+  private void persistAppDef(Path appDir, Service service)
       throws IOException {
-    Path appJson = new Path(appDir, application.getName() + ".json");
-    ServiceApiUtil.jsonSerDeser
-        .save(fs.getFileSystem(), appJson, application, true);
+    Path appJson = new Path(appDir, service.getName() + ".json");
+    jsonSerDeser
+        .save(fs.getFileSystem(), appJson, service, true);
     LOG.info(
-        "Persisted application " + application.getName() + " at " + appJson);
+        "Persisted service " + service.getName() + " at " + appJson);
   }
 
   private void addKeytabResourceIfSecure(SliderFileSystem fileSystem,
       Map<String, LocalResource> localResource, Configuration conf,
-      String appName) throws IOException, BadConfigException {
+      String serviceName) throws IOException, BadConfigException {
     if (!UserGroupInformation.isSecurityEnabled()) {
       return;
     }
@@ -707,7 +737,7 @@ public class ServiceClient extends CompositeService
           conf.get(YarnServiceConf.KEY_AM_LOGIN_KEYTAB_NAME);
       String keytabDir = conf.get(YarnServiceConf.KEY_HDFS_KEYTAB_DIR);
       Path keytabPath =
-          fileSystem.buildKeytabPath(keytabDir, amKeytabName, appName);
+          fileSystem.buildKeytabPath(keytabDir, amKeytabName, serviceName);
       if (fileSystem.getFileSystem().exists(keytabPath)) {
         LocalResource keytabRes =
             fileSystem.createAmResource(keytabPath, LocalResourceType.FILE);
@@ -722,23 +752,22 @@ public class ServiceClient extends CompositeService
         } else {
           LOG.warn("The AM will be "
               + "started without a kerberos authenticated identity. "
-              + "The application is therefore not guaranteed to remain "
+              + "The service is therefore not guaranteed to remain "
               + "operational beyond 24 hours.");
         }
       }
     }
   }
 
-  public String updateLifetime(String appName, long lifetime)
+  public String updateLifetime(String serviceName, long lifetime)
       throws YarnException, IOException {
-    getAppId(appName);
-    ApplicationId currentAppId = cachedAppIds.get(appName);
+    ApplicationId currentAppId = getAppId(serviceName);
     ApplicationReport report = yarnClient.getApplicationReport(currentAppId);
     if (report == null) {
-      throw new YarnException("Application not found for " + appName);
+      throw new YarnException("Service not found for " + serviceName);
     }
     ApplicationId appId = report.getApplicationId();
-    LOG.info("Updating lifetime of an application: appName = " + appName
+    LOG.info("Updating lifetime of an service: serviceName = " + serviceName
         + ", appId = " + appId + ", lifetime = " + lifetime);
     Map<ApplicationTimeoutType, String> map = new HashMap<>();
     String newTimeout =
@@ -748,32 +777,55 @@ public class ServiceClient extends CompositeService
         UpdateApplicationTimeoutsRequest.newInstance(appId, map);
     yarnClient.updateApplicationTimeouts(request);
     LOG.info(
-        "Successfully updated lifetime for an application: appName = " + appName
+        "Successfully updated lifetime for an service: serviceName = " + serviceName
             + ", appId = " + appId + ". New expiry time in ISO8601 format is "
             + newTimeout);
     return newTimeout;
   }
 
-  public Application getStatus(String appName)
+  public ServiceState convertState(FinalApplicationStatus status) {
+    switch (status) {
+    case UNDEFINED:
+      return ServiceState.ACCEPTED;
+    case FAILED:
+    case KILLED:
+      return ServiceState.FAILED;
+    case ENDED:
+    case SUCCEEDED:
+      return ServiceState.STOPPED;
+    }
+    return ServiceState.ACCEPTED;
+  }
+
+  public Service getStatus(String serviceName)
       throws IOException, YarnException {
-    validateClusterName(appName);
-    ApplicationId currentAppId = getAppId(appName);
+    ServiceApiUtil.validateNameFormat(serviceName, getConfig());
+    ApplicationId currentAppId = getAppId(serviceName);
     ApplicationReport appReport = yarnClient.getApplicationReport(currentAppId);
-    ClientAMProtocol amProxy = getAMProxy(appName, appReport);
-    Application appSpec;
-    if (amProxy != null) {
-      GetStatusResponseProto response =
-          amProxy.getStatus(GetStatusRequestProto.newBuilder().build());
-      appSpec = ServiceApiUtil.jsonSerDeser.fromJson(response.getStatus());
-    } else {
-      appSpec = new Application();
-      appSpec.setName(appName);
-    }
+    Service appSpec = new Service();
+    appSpec.setName(serviceName);
+    appSpec.setState(convertState(appReport.getFinalApplicationStatus()));
     ApplicationTimeout lifetime =
         appReport.getApplicationTimeouts().get(ApplicationTimeoutType.LIFETIME);
     if (lifetime != null) {
       appSpec.setLifetime(lifetime.getRemainingTime());
     }
+
+    if (appReport.getYarnApplicationState() != RUNNING) {
+      LOG.info("Service {} is at {} state", serviceName,
+          appReport.getYarnApplicationState());
+      return appSpec;
+    }
+    if (StringUtils.isEmpty(appReport.getHost())) {
+      LOG.warn(serviceName + " AM hostname is empty");
+      return appSpec;
+    }
+    ClientAMProtocol amProxy =
+        createAMProxy(appReport.getHost(), appReport.getRpcPort());
+    GetStatusResponseProto response =
+        amProxy.getStatus(GetStatusRequestProto.newBuilder().build());
+    appSpec = jsonSerDeser.fromJson(response.getStatus());
+
     return appSpec;
   }
 
@@ -815,58 +867,26 @@ public class ServiceClient extends CompositeService
     }
   }
 
-  // Get AMProxy with the appReport provided
-  protected ClientAMProtocol getAMProxy(String appName, ApplicationReport report)
-      throws IOException {
-    if (!cachedAMProxies.containsKey(appName) && !StringUtils
-        .isEmpty(report.getHost())) {
-      insertAMProxy(appName, report.getHost(), report.getRpcPort());
-    }
-    return cachedAMProxies.get(appName);
-  }
-
-  // Get AMProxy without appReport provided - it'll getAppReport from RM
-  protected ClientAMProtocol getAMProxy(String appName)
-      throws IOException, YarnException {
-    ApplicationId currentAppId = getAppId(appName);
-
-    if (cachedAMProxies.containsKey(appName)) {
-      return cachedAMProxies.get(appName);
-    } else {
-      ApplicationReport appReport =
-          yarnClient.getApplicationReport(currentAppId);
-      String host = appReport.getHost();
-      int port = appReport.getRpcPort();
-      if (!StringUtils.isEmpty(host)) {
-        return insertAMProxy(appName, host, port);
-      }
-      return null;
-    }
-  }
-
-  private ClientAMProtocol insertAMProxy(String appName, String host, int port)
+  protected ClientAMProtocol createAMProxy(String host, int port)
       throws IOException {
     InetSocketAddress address =
         NetUtils.createSocketAddrForHost(host, port);
-    ClientAMProtocol amProxy =
-        ClientAMProxy.createProxy(getConfig(), ClientAMProtocol.class,
+    return ClientAMProxy.createProxy(getConfig(), ClientAMProtocol.class,
         UserGroupInformation.getCurrentUser(), rpc, address);
-    cachedAMProxies.put(appName, amProxy);
-    return amProxy;
   }
 
-  private synchronized ApplicationId getAppId(String appName)
+  private synchronized ApplicationId getAppId(String serviceName)
       throws IOException, YarnException {
-    if (cachedAppIds.containsKey(appName)) {
-      return cachedAppIds.get(appName);
+    if (cachedAppIds.containsKey(serviceName)) {
+      return cachedAppIds.get(serviceName);
     }
-    Application persistedApp = ServiceApiUtil.loadApplication(fs, appName);
-    if (persistedApp == null) {
-      throw new YarnException("Application " + appName
+    Service persistedService = ServiceApiUtil.loadService(fs, serviceName);
+    if (persistedService == null) {
+      throw new YarnException("Service " + serviceName
           + " doesn't exist on hdfs. Please check if the app exists in RM");
     }
-    ApplicationId currentAppId = ApplicationId.fromString(persistedApp.getId());
-    cachedAppIds.put(appName, currentAppId);
+    ApplicationId currentAppId = ApplicationId.fromString(persistedService.getId());
+    cachedAppIds.put(serviceName, currentAppId);
     return currentAppId;
   }
 }

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractActionArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractActionArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractArgsDelegate.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractArgsDelegate.java


+ 3 - 3
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractClusterBuildingActionArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/AbstractClusterBuildingActionArgs.java

@@ -34,7 +34,7 @@ import java.util.Map;
 public abstract class AbstractClusterBuildingActionArgs
     extends AbstractActionArgs {
   @Parameter(names = {ARG_APPDEF},
-      description = "Template application definition file in JSON format.")
+      description = "Template service definition file in JSON format.")
   public File appDef;
 
   public File getAppDef() {
@@ -42,11 +42,11 @@ public abstract class AbstractClusterBuildingActionArgs
   }
 
   @Parameter(names = {
-      ARG_QUEUE }, description = "Queue to submit the application")
+      ARG_QUEUE }, description = "Queue to submit the service")
   public String queue;
 
   @Parameter(names = {
-      ARG_LIFETIME }, description = "Lifetime of the application from the time of request")
+      ARG_LIFETIME }, description = "Lifetime of the service from the time of request")
   public long lifetime;
 
   @ParametersDelegate

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionBuildArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionBuildArgs.java


+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionClientArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionClientArgs.java

@@ -40,7 +40,7 @@ public class ActionClientArgs extends AbstractActionArgs {
   public boolean install;
 
   @Parameter(names = {ARG_NAME},
-      description = "The name of the application")
+      description = "The name of the service")
   public String name;
 
   @Parameter(names = {ARG_PACKAGE},

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionCreateArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionCreateArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionDependencyArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionDependencyArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionDestroyArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionDestroyArgs.java


+ 3 - 3
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionExistsArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionExistsArgs.java

@@ -36,14 +36,14 @@ public class ActionExistsArgs extends AbstractActionArgs {
   }
 
   @Parameter(names = {ARG_LIVE},
-             description = "verify that the application is running")
+             description = "verify that the service is running")
   public boolean live;
   
   @Parameter(names = {ARG_STATE},
-             description = "verify that the application is in the specific YARN state")
+             description = "verify that the service is in the specific YARN state")
   public String state = "";
 
   @Parameter(names = {ARG_OUTPUT, ARG_OUTPUT_SHORT},
-      description = "output file for any application report")
+      description = "output file for any service report")
   public File out;
 }

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionFlexArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionFlexArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionFreezeArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionFreezeArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionHelpArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionHelpArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionKDiagArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionKDiagArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionKeytabArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionKeytabArgs.java


+ 2 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionListArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionListArgs.java

@@ -36,7 +36,7 @@ public class ActionListArgs extends AbstractActionArgs {
   }
 
   @Parameter(names = {ARG_LIVE},
-          description = "List only live application instances")
+          description = "List only live service instances")
   public boolean live;
 
   @Parameter(names = {ARG_STATE},
@@ -48,7 +48,7 @@ public class ActionListArgs extends AbstractActionArgs {
   public boolean verbose = false;
 
   @Parameter(names = {ARG_CONTAINERS},
-      description = "List containers of an application instance")
+      description = "List containers of a service instance")
   public boolean containers;
 
   @Parameter(names = {ARG_VERSION},

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionRegistryArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionRegistryArgs.java

@@ -136,7 +136,7 @@ public class ActionRegistryArgs extends AbstractActionArgs {
   public boolean internal;
 
   @Parameter(names = {ARG_USER},
-      description = "the name of the user whose application is being resolved")
+      description = "the name of the user whose service is being resolved")
   public String user;
 
   /**

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionResolveArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionResolveArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionResourceArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionResourceArgs.java


+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionStatusArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionStatusArgs.java

@@ -38,7 +38,7 @@ public class ActionStatusArgs extends AbstractActionArgs {
   public String output;
 
   @Parameter(names = {ARG_LIFETIME},
-      description = "Lifetime of the application from the time of request")
+      description = "Lifetime of the service from the time of request")
   public boolean lifetime;
 
   public String getOutput() {

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionThawArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionThawArgs.java

@@ -45,7 +45,7 @@ public class ActionThawArgs extends AbstractActionArgs implements
   LaunchArgsDelegate launchArgs = new LaunchArgsDelegate();
 
   @Parameter(names = {ARG_LIFETIME},
-      description = "Life time of the application since application started at"
+      description = "Life time of the service since service started at"
           + " running state")
   public long lifetime;
 

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionTokensArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionTokensArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionUpdateArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ActionUpdateArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ArgOps.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ArgOps.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/Arguments.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/Arguments.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ClientArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ClientArgs.java


+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/CommonArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/CommonArgs.java

@@ -124,7 +124,7 @@ public abstract class CommonArgs extends ArgOps implements SliderActions,
       result = helperMessage.toString();
     } else {
       helperMessage.append("\nUsage: service ").append(commandOfInterest);
-      helperMessage.append(serviceArgs.coreAction.getMinParams() > 0 ? " <application>" : "");
+      helperMessage.append(serviceArgs.coreAction.getMinParams() > 0 ? " <service>" : "");
       helperMessage.append("\n");
       for (ParameterDescription paramDesc : serviceArgs.commander.getCommands()
           .get(commandOfInterest).getParameters()) {

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ComponentArgsDelegate.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/ComponentArgsDelegate.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/DontSplitArguments.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/DontSplitArguments.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/LaunchArgsAccessor.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/LaunchArgsAccessor.java


+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/LaunchArgsDelegate.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/LaunchArgsDelegate.java

@@ -41,7 +41,7 @@ public class LaunchArgsDelegate extends WaitArgsDelegate implements
   }
 
   @Parameter(names = {ARG_OUTPUT, ARG_OUTPUT_SHORT},
-      description = "output file for any application report")
+      description = "output file for any service report")
   public File outputFile;
 
   @Override

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/OptionArgsDelegate.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/OptionArgsDelegate.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/PathArgumentConverter.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/PathArgumentConverter.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderAMArgs.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderAMArgs.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderAMCreateAction.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderAMCreateAction.java


+ 2 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderActions.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/SliderActions.java

@@ -60,7 +60,7 @@ public interface SliderActions {
   String DESCRIBE_ACTION_DESTROY =
         "Destroy a stopped service, service must be stopped first before destroying.";
   String DESCRIBE_ACTION_EXISTS =
-            "Probe for an application running";
+            "Probe for a service running";
   String DESCRIBE_ACTION_FLEX = "Flex a service's component by increasing or decreasing the number of containers.";
   String DESCRIBE_ACTION_FREEZE =
               "Stop a running service";
@@ -74,7 +74,7 @@ public interface SliderActions {
                       "Get the status of a service";
   String DESCRIBE_ACTION_THAW =
                         "Start a service with pre-built specification or a previously stopped service";
-  String DESCRIBE_ACTION_CLIENT = "Install the application client in the specified directory or obtain a client keystore or truststore";
+  String DESCRIBE_ACTION_CLIENT = "Install the service client in the specified directory or obtain a client keystore or truststore";
   String DESCRIBE_ACTION_KEYTAB = "Manage a Kerberos keytab file (install, delete, list) in the sub-folder 'keytabs' of the user's Slider base directory";
   String DESCRIBE_ACTION_RESOURCE = "Manage a file (install, delete, list) in the 'resources' sub-folder of the user's Slider base directory";
 

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/WaitArgsDelegate.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/WaitArgsDelegate.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/client/params/WaitTimeAccessor.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/client/params/WaitTimeAccessor.java


+ 14 - 13
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/component/Component.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/Component.java

@@ -26,13 +26,13 @@ import org.apache.hadoop.yarn.client.api.AMRMClient.ContainerRequest;
 import org.apache.hadoop.yarn.client.api.async.AMRMClientAsync;
 import org.apache.hadoop.yarn.event.AsyncDispatcher;
 import org.apache.hadoop.yarn.event.EventHandler;
-import org.apache.hadoop.yarn.service.compinstance.ComponentInstance;
-import org.apache.hadoop.yarn.service.compinstance.ComponentInstanceId;
+import org.apache.hadoop.yarn.service.component.instance.ComponentInstance;
+import org.apache.hadoop.yarn.service.component.instance.ComponentInstanceId;
 import org.apache.hadoop.yarn.service.ContainerFailureTracker;
 import org.apache.hadoop.yarn.service.ServiceContext;
 import org.apache.hadoop.yarn.service.ServiceScheduler;
-import org.apache.hadoop.yarn.service.compinstance.ComponentInstanceEvent;
-import org.apache.hadoop.yarn.service.metrics.ServiceMetrics;
+import org.apache.hadoop.yarn.service.component.instance.ComponentInstanceEvent;
+import org.apache.hadoop.yarn.service.ServiceMetrics;
 import org.apache.hadoop.yarn.state.InvalidStateTransitionException;
 import org.apache.hadoop.yarn.state.MultipleArcTransition;
 import org.apache.hadoop.yarn.state.SingleArcTransition;
@@ -40,8 +40,8 @@ import org.apache.hadoop.yarn.state.StateMachine;
 import org.apache.hadoop.yarn.state.StateMachineFactory;
 import org.apache.hadoop.yarn.util.Apps;
 import org.apache.hadoop.yarn.service.utils.SliderUtils;
-import org.apache.hadoop.yarn.service.servicemonitor.probe.MonitorUtils;
-import org.apache.hadoop.yarn.service.servicemonitor.probe.Probe;
+import org.apache.hadoop.yarn.service.monitor.probe.MonitorUtils;
+import org.apache.hadoop.yarn.service.monitor.probe.Probe;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
@@ -53,13 +53,14 @@ import java.util.LinkedList;
 import java.util.List;
 import java.util.Map;
 import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.atomic.AtomicInteger;
 import java.util.concurrent.atomic.AtomicLong;
 import java.util.concurrent.locks.ReentrantReadWriteLock;
 
 import static org.apache.hadoop.yarn.api.records.ContainerExitStatus.*;
 import static org.apache.hadoop.yarn.service.component.ComponentEventType.*;
-import static org.apache.hadoop.yarn.service.compinstance.ComponentInstanceEventType.STARTED;
-import static org.apache.hadoop.yarn.service.compinstance.ComponentInstanceEventType.STOP;
+import static org.apache.hadoop.yarn.service.component.instance.ComponentInstanceEventType.START;
+import static org.apache.hadoop.yarn.service.component.instance.ComponentInstanceEventType.STOP;
 import static org.apache.hadoop.yarn.service.component.ComponentState.*;
 import static org.apache.hadoop.yarn.service.conf.YarnServiceConf.CONTAINER_FAILURE_THRESHOLD;
 
@@ -85,7 +86,7 @@ public class Component implements EventHandler<ComponentEvent> {
   public int maxContainerFailurePerComp;
   // The number of containers failed since last reset. This excludes preempted,
   // disk_failed containers etc. This will be reset to 0 periodically.
-  public volatile int currentContainerFailure;
+  public AtomicInteger currentContainerFailure = new AtomicInteger(0);
 
   private StateMachine<ComponentState, ComponentEventType, ComponentEvent>
       stateMachine;
@@ -239,7 +240,7 @@ public class Component implements EventHandler<ComponentEvent> {
         ComponentEvent event) {
       component.compInstanceDispatcher.getEventHandler().handle(
           new ComponentInstanceEvent(event.getInstance().getContainerId(),
-              STARTED));
+              START));
       component.incRunningContainers();
       return checkIfStable(component);
     }
@@ -356,7 +357,7 @@ public class Component implements EventHandler<ComponentEvent> {
       String host = scheduler.getLiveInstances().get(status.getContainerId())
           .getNodeId().getHost();
       failureTracker.incNodeFailure(host);
-      currentContainerFailure++ ;
+      currentContainerFailure.getAndIncrement() ;
     }
   }
 
@@ -425,8 +426,8 @@ public class Component implements EventHandler<ComponentEvent> {
 
   public void resetCompFailureCount() {
     LOG.info("[COMPONENT {}]: Reset container failure count from {} to 0.",
-        getName(), currentContainerFailure);
-    currentContainerFailure = 0;
+        getName(), currentContainerFailure.get());
+    currentContainerFailure.set(0);
     failureTracker.resetContainerFailures();
   }
 

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentEvent.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentEvent.java

@@ -21,7 +21,7 @@ package org.apache.hadoop.yarn.service.component;
 import org.apache.hadoop.yarn.api.records.Container;
 import org.apache.hadoop.yarn.api.records.ContainerStatus;
 import org.apache.hadoop.yarn.event.AbstractEvent;
-import org.apache.hadoop.yarn.service.compinstance.ComponentInstance;
+import org.apache.hadoop.yarn.service.component.instance.ComponentInstance;
 
 public class ComponentEvent extends AbstractEvent<ComponentEventType> {
   private long desired;

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentEventType.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentEventType.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentState.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/ComponentState.java


+ 10 - 10
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/compinstance/ComponentInstance.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstance.java

@@ -16,7 +16,7 @@
  * limitations under the License.
  */
 
-package org.apache.hadoop.yarn.service.compinstance;
+package org.apache.hadoop.yarn.service.component.instance;
 
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.Path;
@@ -45,7 +45,7 @@ import org.apache.hadoop.yarn.state.StateMachineFactory;
 import org.apache.hadoop.yarn.util.BoundedAppender;
 import org.apache.hadoop.yarn.service.utils.SliderUtils;
 import org.apache.hadoop.yarn.service.timelineservice.ServiceTimelinePublisher;
-import org.apache.hadoop.yarn.service.servicemonitor.probe.ProbeStatus;
+import org.apache.hadoop.yarn.service.monitor.probe.ProbeStatus;
 import org.apache.hadoop.yarn.service.registry.YarnRegistryViewForProviders;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
@@ -61,8 +61,8 @@ import java.util.concurrent.locks.ReentrantReadWriteLock.WriteLock;
 
 import static org.apache.hadoop.yarn.api.records.ContainerExitStatus.KILLED_BY_APPMASTER;
 import static org.apache.hadoop.yarn.api.records.ContainerState.COMPLETE;
-import static org.apache.hadoop.yarn.service.compinstance.ComponentInstanceEventType.*;
-import static org.apache.hadoop.yarn.service.compinstance.ComponentInstanceState.*;
+import static org.apache.hadoop.yarn.service.component.instance.ComponentInstanceEventType.*;
+import static org.apache.hadoop.yarn.service.component.instance.ComponentInstanceState.*;
 
 public class ComponentInstance implements EventHandler<ComponentInstanceEvent>,
     Comparable<ComponentInstance> {
@@ -95,17 +95,17 @@ public class ComponentInstance implements EventHandler<ComponentInstanceEvent>,
       stateMachineFactory =
       new StateMachineFactory<ComponentInstance, ComponentInstanceState,
           ComponentInstanceEventType, ComponentInstanceEvent>(INIT)
-      .addTransition(INIT, RUNNING_BUT_UNREADY, STARTED,
+      .addTransition(INIT, STARTED, START,
           new ContainerStartedTransition())
 
       //From Running
-      .addTransition(RUNNING_BUT_UNREADY, INIT, STOP,
+      .addTransition(STARTED, INIT, STOP,
           new ContainerStoppedTransition())
-      .addTransition(RUNNING_BUT_UNREADY, READY, BECOME_READY,
+      .addTransition(STARTED, READY, BECOME_READY,
           new ContainerBecomeReadyTransition())
 
       // FROM READY
-      .addTransition(READY, RUNNING_BUT_UNREADY, BECOME_NOT_READY,
+      .addTransition(READY, STARTED, BECOME_NOT_READY,
           new ContainerBecomeNotReadyTransition())
       .addTransition(READY, INIT, STOP, new ContainerStoppedTransition())
       .installTopology();
@@ -201,11 +201,11 @@ public class ComponentInstance implements EventHandler<ComponentInstanceEvent>,
 
       boolean shouldExit = false;
       // check if it exceeds the failure threshold
-      if (comp.currentContainerFailure > comp.maxContainerFailurePerComp) {
+      if (comp.currentContainerFailure.get() > comp.maxContainerFailurePerComp) {
         String exitDiag = MessageFormat.format(
             "[COMPONENT {0}]: Failed {1} times, exceeded the limit - {2}. Shutting down now... "
                 + System.lineSeparator(),
-            comp.getName(), comp.currentContainerFailure, comp.maxContainerFailurePerComp);
+            comp.getName(), comp.currentContainerFailure.get(), comp.maxContainerFailurePerComp);
         compInstance.diagnostics.append(exitDiag);
         // append to global diagnostics that will be reported to RM.
         comp.getScheduler().getDiagnostics().append(containerDiag);

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/compinstance/ComponentInstanceEvent.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstanceEvent.java

@@ -16,7 +16,7 @@
  * limitations under the License.
  */
 
-package org.apache.hadoop.yarn.service.compinstance;
+package org.apache.hadoop.yarn.service.component.instance;
 
 import org.apache.hadoop.yarn.api.records.ContainerId;
 import org.apache.hadoop.yarn.api.records.ContainerStatus;

+ 2 - 3
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/compinstance/ComponentInstanceEventType.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstanceEventType.java

@@ -16,11 +16,10 @@
  * limitations under the License.
  */
 
-package org.apache.hadoop.yarn.service.compinstance;
+package org.apache.hadoop.yarn.service.component.instance;
 
 public enum ComponentInstanceEventType {
-
-  STARTED,
+  START,
   STOP,
   BECOME_READY,
   BECOME_NOT_READY

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/compinstance/ComponentInstanceId.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstanceId.java

@@ -16,7 +16,7 @@
  * limitations under the License.
  */
 
-package org.apache.hadoop.yarn.service.compinstance;
+package org.apache.hadoop.yarn.service.component.instance;
 
 import org.apache.hadoop.yarn.api.records.ContainerId;
 

+ 2 - 2
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/compinstance/ComponentInstanceState.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/component/instance/ComponentInstanceState.java

@@ -16,11 +16,11 @@
  * limitations under the License.
  */
 
-package org.apache.hadoop.yarn.service.compinstance;
+package org.apache.hadoop.yarn.service.component.instance;
 
 public enum ComponentInstanceState {
   INIT,
-  RUNNING_BUT_UNREADY,
+  STARTED,
   READY,
   UPGRADING
 }

+ 6 - 6
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/conf/RestApiConstants.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/conf/RestApiConstants.java

@@ -20,14 +20,14 @@ package org.apache.hadoop.yarn.service.conf;
 public interface RestApiConstants {
 
   // Rest endpoints
-  String CONTEXT_ROOT = "/services/v1";
-  String VERSION = "/version";
-  String APP_ROOT_PATH = "/applications";
-  String APP_PATH = "/applications/{app_name}";
-  String COMPONENT_PATH = "/applications/{app_name}/components/{component_name}";
+  String CONTEXT_ROOT = "/ws/v1";
+  String VERSION = "/services/version";
+  String SERVICE_ROOT_PATH = "/services";
+  String SERVICE_PATH = "/services/{service_name}";
+  String COMPONENT_PATH = "/services/{service_name}/components/{component_name}";
 
   // Query param
-  String APP_NAME = "app_name";
+  String SERVICE_NAME = "service_name";
   String COMPONENT_NAME = "component_name";
 
   String DEFAULT_COMPONENT_NAME = "default";

+ 4 - 4
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/conf/SliderExitCodes.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/conf/SliderExitCodes.java

@@ -48,12 +48,12 @@ public interface SliderExitCodes extends LauncherExitCodes {
   int EXIT_YARN_SERVICE_FINISHED_WITH_ERROR = 68;
 
   /**
-   * the application instance is unknown: {@value}
+   * the service instance is unknown: {@value}
    */
   int EXIT_UNKNOWN_INSTANCE =     69;
 
   /**
-   * the application instance is in the wrong state for that operation: {@value}
+   * the service instance is in the wrong state for that operation: {@value}
    */
   int EXIT_BAD_STATE =            70;
 
@@ -69,13 +69,13 @@ public interface SliderExitCodes extends LauncherExitCodes {
   int EXIT_DEPLOYMENT_FAILED =    72;
 
   /**
-   * The application is live -and the requested operation
+   * The service is live -and the requested operation
    * does not work if the cluster is running
    */
   int EXIT_APPLICATION_IN_USE =   73;
 
   /**
-   * There already is an application instance of that name
+   * There already is an service instance of that name
    * when an attempt is made to create a new instance
    */
   int EXIT_INSTANCE_EXISTS =      75;

+ 4 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/conf/YarnServiceConf.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/conf/YarnServiceConf.java

@@ -36,6 +36,10 @@ public class YarnServiceConf {
 
   public static final String YARN_QUEUE = "yarn.service.queue";
 
+  public static final String API_SERVER_ADDRESS = "yarn.service.api-server.address";
+  public static final String DEFAULT_API_SERVER_ADDRESS = "0.0.0.0:";
+  public static final int DEFAULT_API_SERVER_PORT = 9191;
+
   /**
    * The yarn service base path:
    * Defaults to HomeDir/.yarn/

+ 1 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/conf/YarnServiceConstants.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/conf/YarnServiceConstants.java

@@ -35,7 +35,7 @@ public interface YarnServiceConstants {
   String DEPENDENCY_DIR_PERMISSIONS = "755";
 
   /**
-   * Application type for YARN service
+   * Service type for YARN service
    */
   String APP_TYPE = "yarn-service";
 

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/AbstractLauncher.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/AbstractLauncher.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/ClasspathConstructor.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/ClasspathConstructor.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/CommandLineBuilder.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/CommandLineBuilder.java


+ 8 - 8
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/ContainerLaunchService.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/ContainerLaunchService.java

@@ -22,10 +22,10 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.service.AbstractService;
 import org.apache.hadoop.yarn.api.records.Container;
 import org.apache.hadoop.yarn.service.api.records.Component;
-import org.apache.hadoop.yarn.service.compinstance.ComponentInstance;
+import org.apache.hadoop.yarn.service.component.instance.ComponentInstance;
 import org.apache.hadoop.yarn.service.provider.ProviderService;
 import org.apache.hadoop.yarn.service.provider.ProviderFactory;
-import org.apache.hadoop.yarn.service.api.records.Application;
+import org.apache.hadoop.yarn.service.api.records.Service;
 import org.apache.hadoop.yarn.service.utils.SliderFileSystem;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
@@ -60,23 +60,23 @@ public class ContainerLaunchService extends AbstractService{
     super.serviceStop();
   }
 
-  public void launchCompInstance(Application application,
+  public void launchCompInstance(Service service,
       ComponentInstance instance, Container container) {
     ContainerLauncher launcher =
-        new ContainerLauncher(application, instance, container);
+        new ContainerLauncher(service, instance, container);
     executorService.execute(launcher);
   }
 
   private class ContainerLauncher implements Runnable {
     public final Container container;
-    public final Application application;
+    public final Service service;
     public ComponentInstance instance;
 
     public ContainerLauncher(
-        Application application,
+        Service service,
         ComponentInstance instance, Container container) {
       this.container = container;
-      this.application = application;
+      this.service = service;
       this.instance = instance;
     }
 
@@ -86,7 +86,7 @@ public class ContainerLaunchService extends AbstractService{
           compSpec.getArtifact());
       AbstractLauncher launcher = new AbstractLauncher(fs, null);
       try {
-        provider.buildContainerLaunchContext(launcher, application,
+        provider.buildContainerLaunchContext(launcher, service,
             instance, fs, getConfig());
         instance.getComponent().getScheduler().getNmClient()
             .startContainerAsync(container,

+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/CredentialUtils.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/CredentialUtils.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/JavaCommandLineBuilder.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/containerlaunch/JavaCommandLineBuilder.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadClusterStateException.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadClusterStateException.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadCommandArgumentsException.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadCommandArgumentsException.java


+ 0 - 0
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadConfigException.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/BadConfigException.java


+ 5 - 20
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-slider/hadoop-yarn-slider-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/ErrorStrings.java → hadoop-yarn-project/hadoop-yarn/hadoop-yarn-applications/hadoop-yarn-services/hadoop-yarn-services-core/src/main/java/org/apache/hadoop/yarn/service/exceptions/ErrorStrings.java

@@ -19,18 +19,10 @@
 package org.apache.hadoop.yarn.service.exceptions;
 
 public interface ErrorStrings {
-  String E_UNSTABLE_CLUSTER = "Unstable Application Instance :";
-  String E_CLUSTER_RUNNING = "Application Instance running";
-  String E_ALREADY_EXISTS = "already exists";
-  String PRINTF_E_INSTANCE_ALREADY_EXISTS = "Application Instance \"%s\" already exists and is defined in %s";
-  String PRINTF_E_INSTANCE_DIR_ALREADY_EXISTS = "Application Instance dir already exists: %s";
-  String E_MISSING_PATH = "Missing path ";
-  String E_INCOMPLETE_CLUSTER_SPEC =
-    "Cluster specification is marked as incomplete: ";
-  String E_UNKNOWN_INSTANCE = "Unknown application instance ";
-  String E_DESTROY_CREATE_RACE_CONDITION =
-      "created while it was being destroyed";
-  String E_UNKNOWN_ROLE = "Unknown role ";
+
+  String PRINTF_E_INSTANCE_ALREADY_EXISTS = "Service Instance \"%s\" already exists and is defined in %s";
+  String PRINTF_E_INSTANCE_DIR_ALREADY_EXISTS = "Service Instance dir already exists: %s";
+
   /**
    * ERROR Strings
    */
@@ -46,12 +38,5 @@ public interface ErrorStrings {
   String ERROR_TOO_MANY_ARGUMENTS =
     "Too many arguments";
   String ERROR_DUPLICATE_ENTRY = "Duplicate entry for ";
-  String E_APPLICATION_NOT_RUNNING = "Application not running";
-  String E_FINISHED_APPLICATION = E_APPLICATION_NOT_RUNNING + ": %s state=%s ";
-  String E_NO_IMAGE_OR_HOME_DIR_SPECIFIED =
-    "Neither an image path nor binary home directory were specified";
-  String E_BOTH_IMAGE_AND_HOME_DIR_SPECIFIED =
-    "Both application image path and home dir have been provided";
-  String E_CONFIGURATION_DIRECTORY_NOT_FOUND =
-    "Configuration directory \"%s\" not found";
+
 }

部分文件因为文件数量过多而无法显示