YARN-6613. Update json validation for new native services providers. Contributed by Billie Rinaldi

This commit is contained in:
Jian He 2017-05-25 12:47:19 -07:00
parent 9085cd5c6e
commit a0574e7f4f
46 changed files with 1432 additions and 1013 deletions

View File

@ -28,11 +28,6 @@
<packaging>jar</packaging>
<description>Hadoop YARN REST APIs for services</description>
<properties>
<test.failIfNoTests>false</test.failIfNoTests>
<powermock.version>1.6.5</powermock.version>
</properties>
<build>
<!-- resources are filtered for dynamic updates. This gets build info in-->
@ -81,30 +76,10 @@
<plugin>
<groupId>org.apache.maven.plugins</groupId>
<artifactId>maven-surefire-plugin</artifactId>
<version>${maven-surefire-plugin.version}</version>
<configuration>
<reuseForks>${test.reuseForks}</reuseForks>
<forkMode>${test.forkMode}</forkMode>
<forkCount>1</forkCount>
<forkedProcessTimeoutInSeconds>${test.forkedProcessTimeoutInSeconds}
</forkedProcessTimeoutInSeconds>
<threadCount>1</threadCount>
<argLine>${test.argLine}</argLine>
<failIfNoTests>${test.failIfNoTests}</failIfNoTests>
<redirectTestOutputToFile>${build.redirect.test.output.to.file}</redirectTestOutputToFile>
<environmentVariables>
<PATH>${test.env.path}</PATH>
<JAVA_HOME>${java.home}</JAVA_HOME>
</environmentVariables>
<systemPropertyVariables>
<java.net.preferIPv4Stack>true</java.net.preferIPv4Stack>
<java.awt.headless>true</java.awt.headless>
</systemPropertyVariables>
<includes>
<include>**/Test*.java</include>
</includes>
<excludes>
<exclude>**/Test*$*.java</exclude>
</excludes>
</configuration>
</plugin>
@ -120,13 +95,6 @@
<artifactId>hadoop-yarn-slider-core</artifactId>
<version>${project.version}</version>
</dependency>
<dependency>
<groupId>org.apache.hadoop</groupId>
<artifactId>hadoop-common</artifactId>
<type>test-jar</type>
<scope>test</scope>
<version>${project.version}</version>
</dependency>
<dependency>
<groupId>io.swagger</groupId>
<artifactId>swagger-annotations</artifactId>
@ -147,29 +115,6 @@
<groupId>com.fasterxml.jackson.jaxrs</groupId>
<artifactId>jackson-jaxrs-json-provider</artifactId>
</dependency>
<dependency>
<groupId>org.mockito</groupId>
<artifactId>mockito-all</artifactId>
<scope>test</scope>
</dependency>
<dependency>
<groupId>org.powermock</groupId>
<artifactId>powermock-module-junit4</artifactId>
<version>${powermock.version}</version>
<scope>test</scope>
</dependency>
<dependency>
<groupId>org.powermock</groupId>
<artifactId>powermock-api-easymock</artifactId>
<version>${powermock.version}</version>
<scope>test</scope>
<exclusions>
<exclusion>
<groupId>org.easymock</groupId>
<artifactId>easymock</artifactId>
</exclusion>
</exclusions>
</dependency>
</dependencies>
<profiles>

View File

@ -158,7 +158,7 @@ definitions:
type: string
description: A unique application id.
artifact:
description: Artifact of single-component applications. Mandatory if components attribute is not specified.
description: Artifact of single-component applications.
$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.
@ -230,16 +230,16 @@ definitions:
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.
Artifact:
description: Artifact of an application component.
description: Artifact of an application 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, etc.
description: Artifact id. Examples are package location uri for tarball based apps, image name for docker, name of application, etc.
type:
type: string
description: Artifact type, like docker, tarball, etc. (optional).
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).
enum:
- DOCKER
- TARBALL
@ -269,7 +269,7 @@ definitions:
$ref: '#/definitions/Artifact'
launch_command:
type: string
description: The custom launch command of this component (optional). When specified at the component level, it overrides the value specified at the global level (if any).
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.
$ref: '#/definitions/Resource'
@ -344,7 +344,7 @@ definitions:
- HADOOP_XML
dest_file:
type: string
description: The absolute path that this configuration file should be mounted as, in the application container.
description: The path that this configuration file should be created as. If it is an absolute path, it will be mounted into the DOCKER container. Absolute paths are only allowed for DOCKER containers. If it is a relative path, only the file name should be provided, and the file will be created in the container local working directory under a folder named conf.
src_file:
type: string
description: This provides the source location of the configuration file, the content of which is dumped to dest_file post property substitutions, in the format as specified in type. Typically the src_file would point to a source controlled network accessible file maintained by tools like puppet, chef, or hdfs etc. Currently, only hdfs is supported.

View File

@ -1,209 +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.services.api.impl;
import static org.apache.slider.util.RestApiConstants.*;
import static org.apache.slider.util.RestApiErrorMessages.*;
import java.util.ArrayList;
import org.apache.slider.api.resource.Application;
import org.apache.slider.api.resource.Artifact;
import org.apache.slider.api.resource.Resource;
import org.apache.slider.util.ServiceApiUtil;
import org.junit.After;
import org.junit.Assert;
import org.junit.Before;
import org.junit.Test;
import org.junit.runner.RunWith;
import org.powermock.core.classloader.annotations.SuppressStaticInitializationFor;
import org.powermock.modules.junit4.PowerMockRunner;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
/**
* Test class for application life time monitor feature test.
*/
@RunWith(PowerMockRunner.class)
@SuppressStaticInitializationFor("org.apache.hadoop.yarn.services.api.impl.ApplicationApiService")
public class TestApplicationApiService {
private static final Logger logger = LoggerFactory
.getLogger(TestApplicationApiService.class);
private static String EXCEPTION_PREFIX = "Should have thrown exception: ";
private static String NO_EXCEPTION_PREFIX = "Should not have thrown exception: ";
private ApplicationApiService appApiService;
@Before
public void setup() throws Exception {
appApiService = new ApplicationApiService();
}
@After
public void tearDown() throws Exception {
}
@Test(timeout = 90000)
public void testValidateApplicationPostPayload() throws Exception {
Application app = new Application();
// no name
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(EXCEPTION_PREFIX + "application with no name");
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_APPLICATION_NAME_INVALID, e.getMessage());
}
// bad format name
String[] badNames = { "4finance", "Finance", "finance@home" };
for (String badName : badNames) {
app.setName(badName);
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(EXCEPTION_PREFIX + "application with bad name " + badName);
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_APPLICATION_NAME_INVALID_FORMAT,
e.getMessage());
}
}
// no artifact
app.setName("finance_home");
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(EXCEPTION_PREFIX + "application with no artifact");
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_ARTIFACT_INVALID, e.getMessage());
}
// no artifact id
Artifact artifact = new Artifact();
app.setArtifact(artifact);
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(EXCEPTION_PREFIX + "application with no artifact id");
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_ARTIFACT_ID_INVALID, e.getMessage());
}
// if artifact is of type APPLICATION then everything is valid here
artifact.setType(Artifact.TypeEnum.APPLICATION);
artifact.setId("app.io/hbase:facebook_0.2");
app.setNumberOfContainers(5l);
try {
ServiceApiUtil.validateApplicationPayload(app, null);
} catch (IllegalArgumentException e) {
logger.error("application attributes specified should be valid here", e);
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
// default-component, default-lifetime and the property component_type
// should get assigned here
Assert.assertEquals(app.getComponents().get(0).getName(),
DEFAULT_COMPONENT_NAME);
Assert.assertEquals(app.getLifetime(), DEFAULT_UNLIMITED_LIFETIME);
//TODO handle external app
// unset artifact type, default component and no of containers to test other
// validation logic
artifact.setType(null);
app.setComponents(new ArrayList<>());
app.setNumberOfContainers(null);
// resource not specified
artifact.setId("docker.io/centos:centos7");
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(EXCEPTION_PREFIX + "application with no resource");
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_RESOURCE_INVALID, e.getMessage());
}
// memory not specified
Resource res = new Resource();
app.setResource(res);
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(EXCEPTION_PREFIX + "application with no memory");
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_RESOURCE_MEMORY_INVALID, e.getMessage());
}
// cpu does not need to be always specified, it's an optional feature in yarn
// invalid no of cpus
res.setMemory("100mb");
res.setCpus(-2);
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(
EXCEPTION_PREFIX + "application with invalid no of cpups");
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_RESOURCE_CPUS_INVALID_RANGE, e.getMessage());
}
// number of containers not specified
res.setCpus(2);
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(
EXCEPTION_PREFIX + "application with no container count");
} catch (IllegalArgumentException e) {
Assert.assertTrue(e.getMessage().contains(ERROR_CONTAINERS_COUNT_INVALID));
}
// specifying profile along with cpus/memory raises exception
res.setProfile("hbase_finance_large");
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(EXCEPTION_PREFIX
+ "application with resource profile along with cpus/memory");
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_RESOURCE_PROFILE_MULTIPLE_VALUES_NOT_SUPPORTED,
e.getMessage());
}
// currently resource profile alone is not supported.
// TODO: remove the next test once it is supported.
res.setCpus(null);
res.setMemory(null);
try {
ServiceApiUtil.validateApplicationPayload(app, null);
Assert.fail(EXCEPTION_PREFIX
+ "application with resource profile only - NOT SUPPORTED");
} catch (IllegalArgumentException e) {
Assert.assertEquals(ERROR_RESOURCE_PROFILE_NOT_SUPPORTED_YET,
e.getMessage());
}
// unset profile here and add cpus/memory back
res.setProfile(null);
res.setCpus(2);
res.setMemory("2gb");
// everything valid here
app.setNumberOfContainers(5l);
try {
ServiceApiUtil.validateApplicationPayload(app, null);
} catch (IllegalArgumentException e) {
logger.error("application attributes specified should be valid here", e);
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
// Now test with components
}
}

View File

@ -286,6 +286,10 @@ public class Application extends BaseResource {
this.components = components;
}
public void addComponent(Component component) {
components.add(component);
}
public Component getComponent(String name) {
for (Component component : components) {
if (component.getName().equals(name)) {

View File

@ -50,9 +50,9 @@ public class Component implements Serializable {
private String name = null;
private List<String> dependencies = new ArrayList<String>();
private ReadinessCheck readinessCheck = null;
private Artifact artifact = new Artifact();
private Artifact artifact = null;
private String launchCommand = null;
private Resource resource = new Resource();
private Resource resource = null;
private Long numberOfContainers = null;
private Boolean uniqueComponentSupport = false;
private Boolean runPrivilegedContainer = false;
@ -406,4 +406,41 @@ public class Component implements Serializable {
}
return o.toString().replace("\n", "\n ");
}
/**
* Merge from another component into this component without overwriting.
*/
public void mergeFrom(Component that) {
if (this.getArtifact() == null) {
this.setArtifact(that.getArtifact());
}
if (this.getResource() == null) {
this.setResource(that.getResource());
}
if (this.getNumberOfContainers() == null) {
this.setNumberOfContainers(that.getNumberOfContainers());
}
if (this.getLaunchCommand() == null) {
this.setLaunchCommand(that.getLaunchCommand());
}
this.getConfiguration().mergeFrom(that.getConfiguration());
if (this.getQuicklinks() == null) {
this.setQuicklinks(that.getQuicklinks());
}
if (this.getRunPrivilegedContainer() == null) {
this.setRunPrivilegedContainer(that.getRunPrivilegedContainer());
}
if (this.getUniqueComponentSupport() == null) {
this.setUniqueComponentSupport(that.getUniqueComponentSupport());
}
if (this.getDependencies() == null) {
this.setDependencies(that.getDependencies());
}
if (this.getPlacementPolicy() == null) {
this.setPlacementPolicy(that.getPlacementPolicy());
}
if (this.getReadinessCheck() == null) {
this.setReadinessCheck(that.getReadinessCheck());
}
}
}

View File

@ -22,6 +22,7 @@ import com.fasterxml.jackson.annotation.JsonProperty;
import io.swagger.annotations.ApiModel;
import io.swagger.annotations.ApiModelProperty;
import org.apache.commons.lang.StringUtils;
import org.apache.slider.common.tools.SliderUtils;
import java.io.Serializable;
import java.util.ArrayList;
@ -197,8 +198,10 @@ public class Configuration implements Serializable {
* this ConfigFile.
*/
public synchronized void mergeFrom(Configuration that) {
this.properties.putAll(that.getProperties());
this.env.putAll(that.getEnv());
SliderUtils.mergeMapsIgnoreDuplicateKeys(this.properties, that
.getProperties());
SliderUtils.mergeMapsIgnoreDuplicateKeys(this.env, that.getEnv());
Map<String, ConfigFile> thatMap = new HashMap<>();
for (ConfigFile file : that.getFiles()) {
thatMap.put(file.getDestFile(), file.copy());
@ -206,7 +209,8 @@ public class Configuration implements Serializable {
for (ConfigFile thisFile : files) {
if(thatMap.containsKey(thisFile.getDestFile())) {
ConfigFile thatFile = thatMap.get(thisFile.getDestFile());
thisFile.getProps().putAll(thatFile.getProps());
SliderUtils.mergeMapsIgnoreDuplicateKeys(thisFile.getProps(),
thatFile.getProps());
thatMap.remove(thisFile.getDestFile());
}
}

View File

@ -19,7 +19,6 @@
package org.apache.slider.client;
import com.google.common.annotations.VisibleForTesting;
import com.google.common.io.Files;
import org.apache.commons.lang.ArrayUtils;
import org.apache.commons.lang.StringUtils;
import org.apache.hadoop.conf.Configuration;
@ -101,6 +100,7 @@ import org.apache.slider.common.params.Arguments;
import org.apache.slider.common.params.ClientArgs;
import org.apache.slider.common.params.CommonArgs;
import org.apache.slider.common.tools.ConfigHelper;
import org.apache.slider.common.tools.Duration;
import org.apache.slider.common.tools.SliderFileSystem;
import org.apache.slider.common.tools.SliderUtils;
import org.apache.slider.common.tools.SliderVersionInfo;
@ -142,8 +142,6 @@ import org.apache.zookeeper.KeeperException;
import org.apache.zookeeper.ZooDefs;
import org.apache.zookeeper.data.ACL;
import org.codehaus.jackson.map.PropertyNamingStrategy;
import org.codehaus.jettison.json.JSONException;
import org.codehaus.jettison.json.JSONObject;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@ -636,16 +634,17 @@ public class SliderClient extends AbstractSliderLaunchedService implements RunSe
public int actionBuild(Application application) throws YarnException,
IOException {
Path appDir = checkAppNotExistOnHdfs(application);
ServiceApiUtil.validateAndResolveApplication(application, sliderFileSystem);
persistApp(appDir, application);
deployedClusterName = application.getName();
return EXIT_SUCCESS;
}
public ApplicationId actionCreate(Application application)
throws IOException, YarnException {
ServiceApiUtil.validateApplicationPayload(application,
sliderFileSystem.getFileSystem());
String appName = application.getName();
validateClusterName(appName);
ServiceApiUtil.validateAndResolveApplication(application, sliderFileSystem);
verifyNoLiveApp(appName, "Create");
Path appDir = checkAppNotExistOnHdfs(application);
@ -880,6 +879,14 @@ public class SliderClient extends AbstractSliderLaunchedService implements RunSe
return appDir;
}
private Path checkAppExistOnHdfs(String appName)
throws IOException, SliderException {
Path appDir = sliderFileSystem.buildClusterDirPath(appName);
sliderFileSystem.verifyPathExists(
new Path(appDir, appName + ".json"));
return appDir;
}
private void persistApp(Path appDir, Application application)
throws IOException, SliderException {
FsPermission appDirPermission = new FsPermission("750");
@ -1125,7 +1132,9 @@ public class SliderClient extends AbstractSliderLaunchedService implements RunSe
YarnException,
IOException {
if (clientInfo.install) {
return doClientInstall(clientInfo);
// TODO implement client install
throw new UnsupportedOperationException("Client install not yet " +
"supported");
} else {
throw new BadCommandArgumentsException(
"Only install, keystore, and truststore commands are supported for the client.\n"
@ -1134,66 +1143,6 @@ public class SliderClient extends AbstractSliderLaunchedService implements RunSe
}
}
private int doClientInstall(ActionClientArgs clientInfo)
throws IOException, SliderException {
require(clientInfo.installLocation != null,
E_INVALID_INSTALL_LOCATION +"\n"
+ CommonArgs.usage(serviceArgs, ACTION_CLIENT));
require(clientInfo.installLocation.exists(),
E_INSTALL_PATH_DOES_NOT_EXIST + ": " + clientInfo.installLocation.getAbsolutePath());
require(clientInfo.installLocation.isDirectory(),
E_INVALID_INSTALL_PATH + ": " + clientInfo.installLocation.getAbsolutePath());
File pkgFile;
File tmpDir = null;
require(isSet(clientInfo.packageURI) || isSet(clientInfo.name),
E_INVALID_APPLICATION_PACKAGE_LOCATION);
if (isSet(clientInfo.packageURI)) {
pkgFile = new File(clientInfo.packageURI);
} else {
Path appDirPath = sliderFileSystem.buildAppDefDirPath(clientInfo.name);
Path appDefPath = new Path(appDirPath, SliderKeys.DEFAULT_APP_PKG);
require(sliderFileSystem.isFile(appDefPath),
E_INVALID_APPLICATION_PACKAGE_LOCATION);
tmpDir = Files.createTempDir();
pkgFile = new File(tmpDir, SliderKeys.DEFAULT_APP_PKG);
sliderFileSystem.copyHdfsFileToLocal(appDefPath, pkgFile);
}
require(pkgFile.isFile(),
E_UNABLE_TO_READ_SUPPLIED_PACKAGE_FILE + " at %s", pkgFile.getAbsolutePath());
JSONObject config = null;
if(clientInfo.clientConfig != null) {
try {
byte[] encoded = Files.toByteArray(clientInfo.clientConfig);
config = new JSONObject(new String(encoded, Charset.defaultCharset()));
} catch (JSONException jsonEx) {
log.error("Unable to read supplied configuration at {}: {}",
clientInfo.clientConfig, jsonEx);
log.debug("Unable to read supplied configuration at {}: {}",
clientInfo.clientConfig, jsonEx, jsonEx);
throw new BadConfigException(E_MUST_BE_A_VALID_JSON_FILE, jsonEx);
}
}
// TODO handle client install
// Only INSTALL is supported
// ClientProvider
// provider = createClientProvider(SliderProviderFactory.DEFAULT_CLUSTER_TYPE);
// provider.processClientOperation(sliderFileSystem,
// getRegistryOperations(),
// getConfig(),
// "INSTALL",
// clientInfo.installLocation,
// pkgFile,
// config,
// clientInfo.name);
return EXIT_SUCCESS;
}
@Override
public int actionUpdate(String clustername,
AbstractClusterBuildingActionArgs buildInfo) throws
@ -1802,23 +1751,24 @@ public class SliderClient extends AbstractSliderLaunchedService implements RunSe
public int actionStart(String appName, ActionThawArgs thaw)
throws YarnException, IOException {
validateClusterName(appName);
Path appDir = checkAppExistOnHdfs(appName);
Application application = ServiceApiUtil.loadApplication(sliderFileSystem,
appName);
ServiceApiUtil.validateAndResolveApplication(application, sliderFileSystem);
// see if it is actually running and bail out;
verifyNoLiveApp(appName, "Thaw");
Path appDir = sliderFileSystem.buildClusterDirPath(appName);
Path appJson = new Path(appDir, appName + ".json");
Application application =
jsonSerDeser.load(sliderFileSystem.getFileSystem(), appJson);
submitApp(application);
ApplicationId appId = submitApp(application);
application.setId(appId.toString());
// write app definition on to hdfs
persistApp(appDir, application);
return 0;
}
public Map<String, Long> flex(String appName, Map<String, Long>
componentCounts) throws YarnException, IOException {
validateClusterName(appName);
Path appDir = sliderFileSystem.buildClusterDirPath(appName);
Path appJson = new Path(appDir, appName + ".json");
Application persistedApp =
jsonSerDeser.load(sliderFileSystem.getFileSystem(), appJson);
Application persistedApp = ServiceApiUtil.loadApplication(sliderFileSystem,
appName);
Map<String, Long> original = new HashMap<>(componentCounts.size());
for (Component persistedComp : persistedApp.getComponents()) {
String name = persistedComp.getName();
@ -1833,7 +1783,8 @@ public class SliderClient extends AbstractSliderLaunchedService implements RunSe
+ " do not exist in app definition.");
}
jsonSerDeser
.save(sliderFileSystem.getFileSystem(), appJson, persistedApp, true);
.save(sliderFileSystem.getFileSystem(), ServiceApiUtil.getAppJsonPath(
sliderFileSystem, appName), persistedApp, true);
log.info("Updated app definition file for components " + componentCounts
.keySet());
@ -2705,6 +2656,12 @@ public class SliderClient extends AbstractSliderLaunchedService implements RunSe
yarnClient);
}
@VisibleForTesting
public ApplicationReport monitorAppToRunning(Duration duration)
throws YarnException, IOException {
return yarnClient.monitorAppToState(applicationId, YarnApplicationState
.RUNNING, duration);
}
}

View File

@ -245,4 +245,65 @@ public class SliderYarnClientImpl extends YarnClientImpl {
}
return results;
}
/**
* Monitor the submitted application for reaching the requested state.
* Will also report if the app reaches a later state (failed, killed, etc)
* Kill application if duration!= null & time expires.
* @param appId Application Id of application to be monitored
* @param duration how long to wait -must be more than 0
* @param desiredState desired state.
* @return the application report -null on a timeout
* @throws YarnException
* @throws IOException
*/
public ApplicationReport monitorAppToState(
ApplicationId appId, YarnApplicationState desiredState, Duration duration)
throws YarnException, IOException {
if (appId == null) {
throw new BadCommandArgumentsException("null application ID");
}
if (duration.limit <= 0) {
throw new BadCommandArgumentsException("Invalid monitoring duration");
}
log.debug("Waiting {} millis for app to reach state {} ",
duration.limit,
desiredState);
duration.start();
try {
while (true) {
// Get application report for the appId we are interested in
ApplicationReport r = getApplicationReport(appId);
log.debug("queried status is\n{}",
new SliderUtils.OnDemandReportStringifier(r));
YarnApplicationState state = r.getYarnApplicationState();
if (state.ordinal() >= desiredState.ordinal()) {
log.debug("App in desired state (or higher) :{}", state);
return r;
}
if (duration.getLimitExceeded()) {
log.debug(
"Wait limit of {} millis to get to state {}, exceeded; app " +
"status\n {}",
duration.limit,
desiredState,
new SliderUtils.OnDemandReportStringifier(r));
return null;
}
// sleep 1s.
try {
Thread.sleep(1000);
} catch (InterruptedException ignored) {
log.debug("Thread sleep in monitoring loop interrupted");
}
}
} finally {
duration.close();
}
}
}

View File

@ -18,10 +18,6 @@
package org.apache.slider.common;
import java.util.Arrays;
import java.util.Collections;
import java.util.List;
/**
* Keys and various constants for Slider
*/
@ -72,18 +68,11 @@ public interface SliderKeys extends SliderXmlConfKeys {
*/
String APP_TYPE = "org-apache-slider";
/**
* Key for component type. This MUST NOT be set in app_config/global {@value}
*/
String COMPONENT_TYPE_KEY = "site.global.component_type";
/**
* A component type for an external app that has been predefined using the
* slider build command
*/
String COMPONENT_TYPE_EXTERNAL_APP = "external_app";
String COMPONENT_SEPARATOR = "-";
List<String> COMPONENT_KEYS_TO_SKIP = Collections.unmodifiableList(Arrays
.asList("zookeeper.", "env.MALLOC_ARENA_MAX", "site.fs.", "site.dfs."));
/**
* A component type for a client component
@ -91,48 +80,19 @@ public interface SliderKeys extends SliderXmlConfKeys {
String COMPONENT_TYPE_CLIENT = "client";
/**
* Key for application version. This must be set in app_config/global {@value}
* Key for application version.
*/
String APP_VERSION = "site.global.app_version";
String APP_VERSION_UNKNOWN = "awaiting heartbeat...";
/**
* Keys for application container specific properties, like release timeout
*/
String APP_CONTAINER_RELEASE_TIMEOUT = "site.global.app_container.release_timeout_secs";
int APP_CONTAINER_HEARTBEAT_INTERVAL_SEC = 10; // look for HEARTBEAT_IDDLE_INTERVAL_SEC
/**
* JVM arg to force IPv4 {@value}
* Subdirectories of HDFS cluster dir.
*/
String JVM_ENABLE_ASSERTIONS = "-ea";
/**
* JVM arg enable JVM system/runtime {@value}
*/
String JVM_ENABLE_SYSTEM_ASSERTIONS = "-esa";
/**
* JVM arg to force IPv4 {@value}
*/
String JVM_FORCE_IPV4 = "-Djava.net.preferIPv4Stack=true";
/**
* JVM arg to go headless {@value}
*/
String JVM_JAVA_HEADLESS = "-Djava.awt.headless=true";
/**
* This is the name of the dir/subdir containing
* the hbase conf that is propagated via YARN
* {@value}
*/
String PROPAGATED_CONF_DIR_NAME = "propagatedconf";
String INFRA_DIR_NAME = "infra";
String GENERATED_CONF_DIR_NAME = "generated";
String SNAPSHOT_CONF_DIR_NAME = "snapshot";
String DATA_DIR_NAME = "database";
String DATA_DIR_NAME = "data";
String HISTORY_DIR_NAME = "history";
String HISTORY_FILENAME_SUFFIX = "json";
String HISTORY_FILENAME_PREFIX = "rolehistory-";
@ -159,14 +119,6 @@ public interface SliderKeys extends SliderXmlConfKeys {
String CLUSTER_DIRECTORY = "cluster";
String PACKAGE_DIRECTORY = "package";
/**
* JVM property to define the slider configuration directory;
* this is set by the slider script: {@value}
*/
String PROPERTY_CONF_DIR = "slider.confdir";
/**
* JVM property to define the slider lib directory;
* this is set by the slider script: {@value}
@ -183,11 +135,6 @@ public interface SliderKeys extends SliderXmlConfKeys {
*/
String LOG4J_SERVER_PROP_FILENAME = "slideram-log4j.properties";
/**
* Standard log4j file name : {@value}
*/
String LOG4J_PROP_FILENAME = "log4j.properties";
/**
* Log4j sysprop to name the resource :{@value}
*/
@ -209,9 +156,7 @@ public interface SliderKeys extends SliderXmlConfKeys {
*/
String SLIDER_SERVER_XML = "slider-server.xml";
String TMP_LOGDIR_PREFIX = "/tmp/slider-";
String TMP_DIR_PREFIX = "tmp";
String AM_DIR_PREFIX = "appmaster";
/**
* Store the default app definition, e.g. metainfo file or content of a folder
@ -223,53 +168,11 @@ public interface SliderKeys extends SliderXmlConfKeys {
String ADDONS_DIR = "addons";
String SLIDER_JAR = "slider-core.jar";
String JCOMMANDER_JAR = "jcommander.jar";
String GSON_JAR = "gson.jar";
String DEFAULT_APP_PKG = "appPkg.zip";
String DEFAULT_JVM_HEAP = "256M";
int DEFAULT_YARN_MEMORY = 256;
String STDOUT_AM = "slider-out.txt";
String STDERR_AM = "slider-err.txt";
String DEFAULT_GC_OPTS = "";
String HADOOP_USER_NAME = "HADOOP_USER_NAME";
String HADOOP_PROXY_USER = "HADOOP_PROXY_USER";
String SLIDER_PASSPHRASE = "SLIDER_PASSPHRASE";
boolean PROPAGATE_RESOURCE_OPTION = true;
/**
* Security associated keys.
*/
String SECURITY_DIR = "security";
String CRT_FILE_NAME = "ca.crt";
String CSR_FILE_NAME = "ca.csr";
String KEY_FILE_NAME = "ca.key";
String KEYSTORE_FILE_NAME = "keystore.p12";
String CRT_PASS_FILE_NAME = "pass.txt";
String PASS_LEN = "50";
String COMP_STORES_REQUIRED_KEY =
"slider.component.security.stores.required";
String COMP_KEYSTORE_PASSWORD_PROPERTY_KEY =
"slider.component.keystore.password.property";
String COMP_KEYSTORE_PASSWORD_ALIAS_KEY =
"slider.component.keystore.credential.alias.property";
String COMP_KEYSTORE_PASSWORD_ALIAS_DEFAULT =
"component.keystore.credential.alias";
String COMP_TRUSTSTORE_PASSWORD_PROPERTY_KEY =
"slider.component.truststore.password.property";
String COMP_TRUSTSTORE_PASSWORD_ALIAS_KEY =
"slider.component.truststore.credential.alias.property";
String COMP_TRUSTSTORE_PASSWORD_ALIAS_DEFAULT =
"component.truststore.credential.alias";
/**
* Python specific
*/
String PYTHONPATH = "PYTHONPATH";
/**
* Name of the AM filter to use: {@value}
@ -277,34 +180,11 @@ public interface SliderKeys extends SliderXmlConfKeys {
String AM_FILTER_NAME =
"org.apache.hadoop.yarn.server.webproxy.amfilter.AmFilterInitializer";
/**
* Allowed port range. This MUST be set in app_conf/global.
* {@value}
*/
String KEY_ALLOWED_PORT_RANGE = "site.global.slider.allowed.ports";
/**
* env var for custom JVM options.
*/
String SLIDER_JVM_OPTS = "SLIDER_JVM_OPTS";
String SLIDER_CLASSPATH_EXTRA = "SLIDER_CLASSPATH_EXTRA";
String YARN_CONTAINER_PATH = "/node/container/";
String GLOBAL_CONFIG_TAG = "global";
String SYSTEM_CONFIGS = "system_configs";
String JAVA_HOME = "java_home";
String TWO_WAY_SSL_ENABLED = "ssl.server.client.auth";
String INFRA_RUN_SECURITY_DIR = "infra/run/security/";
String CERT_FILE_LOCALIZATION_PATH = INFRA_RUN_SECURITY_DIR + "ca.crt";
String AM_CONFIG_GENERATION = "am.config.generation";
String APP_CONF_DIR = "conf";
String APP_RESOURCES = "application.resources";
String APP_RESOURCES_DIR = "app/resources";
String APP_INSTALL_DIR = "app/install";
String APP_LIB_DIR = "lib";
String OUT_FILE = "stdout.txt";
String ERR_FILE = "stderr.txt";

View File

@ -20,7 +20,6 @@ package org.apache.slider.common.tools;
import com.google.common.base.Preconditions;
import org.apache.commons.lang.StringUtils;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.CommonConfigurationKeys;
import org.apache.hadoop.fs.FSDataInputStream;
@ -43,20 +42,15 @@ import org.apache.slider.core.exceptions.ErrorStrings;
import org.apache.slider.core.exceptions.SliderException;
import org.apache.slider.core.exceptions.UnknownApplicationInstanceException;
import org.apache.slider.core.persist.Filenames;
import org.apache.slider.core.persist.InstancePaths;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import java.io.File;
import java.io.FilenameFilter;
import java.io.FileNotFoundException;
import java.io.IOException;
import java.nio.charset.Charset;
import java.util.HashMap;
import java.util.Map;
import java.util.Enumeration;
import java.util.zip.ZipEntry;
import java.util.zip.ZipFile;
import static org.apache.slider.common.SliderXmlConfKeys.CLUSTER_DIRECTORY_PERMISSIONS;
import static org.apache.slider.common.SliderXmlConfKeys.DEFAULT_CLUSTER_DIRECTORY_PERMISSIONS;
@ -152,34 +146,6 @@ public class CoreFileSystem {
return new Path(path, SliderKeys.ADDONS_DIR + "/" + addonId);
}
/**
* Build up the path string for package install location -no attempt to
* create the directory is made
*
* @return the path for persistent app package
*/
public Path buildPackageDirPath(String packageName, String packageVersion) {
Preconditions.checkNotNull(packageName);
Path path = getBaseApplicationPath();
path = new Path(path, SliderKeys.PACKAGE_DIRECTORY + "/" + packageName);
if (SliderUtils.isSet(packageVersion)) {
path = new Path(path, packageVersion);
}
return path;
}
/**
* Build up the path string for package install location -no attempt to
* create the directory is made
*
* @return the path for persistent app package
*/
public Path buildClusterSecurityDirPath(String clusterName) {
Preconditions.checkNotNull(clusterName);
Path path = buildClusterDirPath(clusterName);
return new Path(path, SliderKeys.SECURITY_DIR);
}
/**
* Build up the path string for keytab install location -no attempt to
* create the directory is made
@ -389,36 +355,6 @@ public class CoreFileSystem {
return isFile;
}
/**
* Create the application-instance specific temporary directory
* in the DFS
*
* @param clustername name of the cluster
* @param subdir application ID
* @return the path; this directory will already have been created
*/
public Path createAppInstanceTempPath(String clustername, String subdir)
throws IOException {
Path tmp = getTempPathForCluster(clustername);
Path instancePath = new Path(tmp, subdir);
fileSystem.mkdirs(instancePath);
return instancePath;
}
/**
* Create the application-instance specific temporary directory
* in the DFS
*
* @param clustername name of the cluster
* @return the path; this directory will already have been deleted
*/
public Path purgeAppInstanceTempFiles(String clustername) throws
IOException {
Path tmp = getTempPathForCluster(clustername);
fileSystem.delete(tmp, true);
return tmp;
}
/**
* Get the base path
*

View File

@ -48,8 +48,6 @@ import org.apache.hadoop.yarn.client.api.AMRMClient;
import org.apache.hadoop.yarn.conf.YarnConfiguration;
import org.apache.slider.Slider;
import org.apache.slider.api.RoleKeys;
import org.apache.slider.api.resource.Application;
import org.apache.slider.api.resource.Component;
import org.apache.slider.api.types.ContainerInformation;
import org.apache.slider.common.SliderKeys;
import org.apache.slider.common.SliderXmlConfKeys;
@ -2540,14 +2538,4 @@ public final class SliderUtils {
long totalMinutes = days * 24 * 60 + hours * 24 + minutes;
return totalMinutes * 60 + seconds;
}
public static void resolve(Application application) {
org.apache.slider.api.resource.Configuration global = application
.getConfiguration();
for (Component component : application.getComponents()) {
mergeMapsIgnoreDuplicateKeys(component.getConfiguration().getProperties(),
global.getProperties());
}
// TODO merge other information to components
}
}
}

View File

@ -1,58 +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.slider.core.persist;
import org.apache.hadoop.fs.Path;
import org.apache.slider.common.SliderKeys;
/**
* Build up all the paths of an instance relative to the supplied instance
* directory.
*/
public class InstancePaths {
public final Path instanceDir;
public final Path snapshotConfPath;
public final Path generatedConfPath;
public final Path historyPath;
public final Path dataPath;
public final Path tmpPath;
public final Path tmpPathAM;
public final Path appDefPath;
public final Path addonsPath;
public InstancePaths(Path instanceDir) {
this.instanceDir = instanceDir;
snapshotConfPath =
new Path(instanceDir, SliderKeys.SNAPSHOT_CONF_DIR_NAME);
generatedConfPath =
new Path(instanceDir, SliderKeys.GENERATED_CONF_DIR_NAME);
historyPath = new Path(instanceDir, SliderKeys.HISTORY_DIR_NAME);
dataPath = new Path(instanceDir, SliderKeys.DATA_DIR_NAME);
tmpPath = new Path(instanceDir, SliderKeys.TMP_DIR_PREFIX);
tmpPathAM = new Path(tmpPath, SliderKeys.AM_DIR_PREFIX);
appDefPath = new Path(tmpPath, SliderKeys.APP_DEF_DIR);
addonsPath = new Path(tmpPath, SliderKeys.ADDONS_DIR);
}
@Override
public String toString() {
return "instance at " + instanceDir;
}
}

View File

@ -18,8 +18,10 @@
package org.apache.slider.providers;
import org.apache.commons.lang.StringUtils;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.hadoop.registry.client.api.RegistryOperations;
import org.apache.slider.api.resource.Artifact;
import org.apache.slider.api.resource.ConfigFile;
@ -30,6 +32,7 @@ import org.codehaus.jettison.json.JSONObject;
import java.io.File;
import java.io.IOException;
import java.nio.file.Paths;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
@ -76,12 +79,50 @@ public abstract class AbstractClientProvider {
/**
* Validate the config files.
* @param configFiles config file list
* @param fileSystem file system
* @param fs file system
*/
public void validateConfigFiles(List<ConfigFile> configFiles, FileSystem
fileSystem) throws IOException {
for (ConfigFile configFile : configFiles) {
validateConfigFile(configFile, fileSystem);
public void validateConfigFiles(List<ConfigFile> configFiles,
FileSystem fs) throws IOException {
Set<String> destFileSet = new HashSet<>();
for (ConfigFile file : configFiles) {
if (file.getType() == null) {
throw new IllegalArgumentException("File type is empty");
}
if (file.getType().equals(ConfigFile.TypeEnum.TEMPLATE) && StringUtils
.isEmpty(file.getSrcFile())) {
throw new IllegalArgumentException(
"Src_file is empty for " + ConfigFile.TypeEnum.TEMPLATE);
}
if (!StringUtils.isEmpty(file.getSrcFile())) {
Path p = new Path(file.getSrcFile());
if (!fs.exists(p)) {
throw new IllegalArgumentException(
"Src_file does not exist for config file: " + file
.getSrcFile());
}
}
if (StringUtils.isEmpty(file.getDestFile())) {
throw new IllegalArgumentException("Dest_file is empty.");
}
if (destFileSet.contains(file.getDestFile())) {
throw new IllegalArgumentException(
"Duplicated ConfigFile exists: " + file.getDestFile());
}
destFileSet.add(file.getDestFile());
java.nio.file.Path destPath = Paths.get(file.getDestFile());
if (!destPath.isAbsolute() && destPath.getNameCount() > 1) {
throw new IllegalArgumentException("Non-absolute dest_file has more " +
"than one path element");
}
// provider-specific validation
validateConfigFile(file, fs);
}
}

View File

@ -22,7 +22,6 @@ import org.apache.slider.api.resource.Artifact;
import org.apache.slider.core.exceptions.SliderException;
import org.apache.slider.providers.docker.DockerProviderFactory;
import org.apache.slider.providers.tarball.TarballProviderFactory;
import org.apache.slider.util.RestApiErrorMessages;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@ -30,7 +29,7 @@ import org.slf4j.LoggerFactory;
* Base class for factories.
*/
public abstract class SliderProviderFactory {
protected static final Logger log =
protected static final Logger LOG =
LoggerFactory.getLogger(SliderProviderFactory.class);
protected SliderProviderFactory() {}
@ -58,10 +57,10 @@ public abstract class SliderProviderFactory {
public static synchronized SliderProviderFactory createSliderProviderFactory(
Artifact artifact) {
if (artifact == null || artifact.getType() == null) {
log.info("Loading service provider type default");
LOG.debug("Loading service provider type default");
return DefaultProviderFactory.getInstance();
}
log.info("Loading service provider type {}", artifact.getType());
LOG.debug("Loading service provider type {}", artifact.getType());
switch (artifact.getType()) {
// TODO add handling for custom types?
// TODO handle application
@ -70,8 +69,9 @@ public abstract class SliderProviderFactory {
case TARBALL:
return TarballProviderFactory.getInstance();
default:
throw new IllegalArgumentException(
RestApiErrorMessages.ERROR_ARTIFACT_INVALID);
throw new IllegalArgumentException(String.format("Resolution error, " +
"%s should not be passed to createSliderProviderFactory",
artifact.getType()));
}
}
}

View File

@ -45,6 +45,6 @@ public class TarballProviderService extends AbstractProviderService {
LocalResourceType type = LocalResourceType.ARCHIVE;
LocalResource packageResource = fileSystem.createAmResource(
artifact, type);
launcher.addLocalResource(APP_INSTALL_DIR, packageResource);
launcher.addLocalResource(APP_LIB_DIR, packageResource);
}
}

View File

@ -107,7 +107,6 @@ import org.apache.slider.core.main.ExitCodeProvider;
import org.apache.slider.core.main.LauncherExitCodes;
import org.apache.slider.core.main.RunService;
import org.apache.slider.core.main.ServiceLauncher;
import org.apache.slider.core.persist.JsonSerDeser;
import org.apache.slider.core.registry.info.CustomRegistryConstants;
import org.apache.slider.providers.ProviderCompleted;
import org.apache.slider.providers.ProviderService;
@ -157,7 +156,7 @@ import org.apache.slider.server.services.workflow.ServiceThreadFactory;
import org.apache.slider.server.services.workflow.WorkflowExecutorService;
import org.apache.slider.server.services.workflow.WorkflowRpcService;
import org.apache.slider.server.services.yarnregistry.YarnRegistryViewForProviders;
import org.codehaus.jackson.map.PropertyNamingStrategy;
import org.apache.slider.util.ServiceApiUtil;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@ -389,9 +388,6 @@ public class SliderAppMaster extends AbstractSliderLaunchedService
*/
private boolean securityEnabled;
private ContentCache contentCache;
private static final JsonSerDeser<Application> jsonSerDeser =
new JsonSerDeser<Application>(Application.class,
PropertyNamingStrategy.CAMEL_CASE_TO_LOWER_CASE_WITH_UNDERSCORES);
/**
* resource limits
@ -590,9 +586,7 @@ public class SliderAppMaster extends AbstractSliderLaunchedService
Path appDir = new Path((serviceArgs.getAppDefDir()));
SliderFileSystem fs = getClusterFS();
fs.setAppDir(appDir);
Path appJson = new Path(appDir, appName + ".json");
log.info("Loading application definition from " + appJson);
application = jsonSerDeser.load(fs.getFileSystem(), appJson);
application = ServiceApiUtil.loadApplication(fs, appName);
log.info("Application Json: " + application);
stateForProviders.setApplicationName(appName);
Configuration serviceConf = getConfig();
@ -821,7 +815,8 @@ public class SliderAppMaster extends AbstractSliderLaunchedService
binding.releaseSelector = new MostRecentContainerReleaseSelector();
binding.nodeReports = nodeReports;
binding.application = application;
binding.serviceHdfsDir = fs.buildClusterDirPath(appName).toString();
binding.serviceHdfsDir = new Path(fs.buildClusterDirPath(appName),
SliderKeys.DATA_DIR_NAME).toString();
appState.buildInstance(binding);
// build up environment variables that the AM wants set in every container
@ -874,11 +869,6 @@ public class SliderAppMaster extends AbstractSliderLaunchedService
scheduleFailureWindowResets(application.getConfiguration());
scheduleEscalation(application.getConfiguration());
for (Component component : application.getComponents()) {
// Merge app-level configuration into component level configuration
component.getConfiguration().mergeFrom(application.getConfiguration());
}
try {
// schedule YARN Registry registration
queue(new ActionRegisterServiceInstance(appName, appid, application));

View File

@ -21,7 +21,7 @@ public interface RestApiErrorMessages {
String ERROR_APPLICATION_NAME_INVALID =
"Application name is either empty or not provided";
String ERROR_APPLICATION_NAME_INVALID_FORMAT =
"Application name is not valid - only lower case letters, digits,"
"Application name %s is not valid - only lower case letters, digits,"
+ " underscore and hyphen are allowed";
String ERROR_APPLICATION_NOT_RUNNING = "Application not running";
@ -76,7 +76,7 @@ public interface RestApiErrorMessages {
String ERROR_ABSENT_NUM_OF_INSTANCE =
"Num of instances should appear either globally or per component";
String ERROR_ABSENT_LAUNCH_COMMAND =
"launch command should appear if type is slider-zip or none";
"Launch_command is required when type is not DOCKER";
String ERROR_QUICKLINKS_FOR_COMP_INVALID = "Quicklinks specified at"
+ " component level, needs corresponding values set at application level";

View File

@ -25,119 +25,159 @@ import org.apache.hadoop.fs.Path;
import org.apache.slider.api.resource.Application;
import org.apache.slider.api.resource.Artifact;
import org.apache.slider.api.resource.Component;
import org.apache.slider.api.resource.ConfigFile;
import org.apache.slider.api.resource.Configuration;
import org.apache.slider.api.resource.Resource;
import org.apache.slider.common.tools.SliderFileSystem;
import org.apache.slider.common.tools.SliderUtils;
import org.apache.slider.core.persist.JsonSerDeser;
import org.apache.slider.providers.AbstractClientProvider;
import org.apache.slider.providers.SliderProviderFactory;
import org.codehaus.jackson.map.PropertyNamingStrategy;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import java.io.IOException;
import java.nio.file.Paths;
import java.util.ArrayList;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
public class ServiceApiUtil {
private static final Logger log =
private static final Logger LOG =
LoggerFactory.getLogger(ServiceApiUtil.class);
private static JsonSerDeser<Application> jsonSerDeser =
new JsonSerDeser<>(Application.class,
PropertyNamingStrategy.CAMEL_CASE_TO_LOWER_CASE_WITH_UNDERSCORES);
@VisibleForTesting
public static void validateApplicationPayload(Application application,
FileSystem fs) throws IOException {
public static void setJsonSerDeser(JsonSerDeser jsd) {
jsonSerDeser = jsd;
}
@VisibleForTesting
public static void validateAndResolveApplication(Application application,
SliderFileSystem fs) throws IOException {
if (StringUtils.isEmpty(application.getName())) {
throw new IllegalArgumentException(
RestApiErrorMessages.ERROR_APPLICATION_NAME_INVALID);
}
if (!SliderUtils.isClusternameValid(application.getName())) {
throw new IllegalArgumentException(
RestApiErrorMessages.ERROR_APPLICATION_NAME_INVALID_FORMAT);
throw new IllegalArgumentException(String.format(
RestApiErrorMessages.ERROR_APPLICATION_NAME_INVALID_FORMAT,
application.getName()));
}
// If the application has no components do top-level checks
if (!hasComponent(application)) {
// artifact
if (application.getArtifact() == null) {
throw new IllegalArgumentException(
RestApiErrorMessages.ERROR_ARTIFACT_INVALID);
}
if (StringUtils.isEmpty(application.getArtifact().getId())) {
throw new IllegalArgumentException(
RestApiErrorMessages.ERROR_ARTIFACT_ID_INVALID);
// If artifact is of type APPLICATION, read other application components
if (application.getArtifact() != null && application.getArtifact()
.getType() == Artifact.TypeEnum.APPLICATION) {
if (StringUtils.isEmpty(application.getArtifact().getId())) {
throw new IllegalArgumentException(
RestApiErrorMessages.ERROR_ARTIFACT_ID_INVALID);
}
Application otherApplication = loadApplication(fs,
application.getArtifact().getId());
application.setComponents(otherApplication.getComponents());
application.setArtifact(null);
SliderUtils.mergeMapsIgnoreDuplicateKeys(application.getQuicklinks(),
otherApplication.getQuicklinks());
} else {
// Since it is a simple app with no components, create a default
// component
Component comp = createDefaultComponent(application);
validateComponent(comp, fs.getFileSystem());
application.getComponents().add(comp);
if (application.getLifetime() == null) {
application.setLifetime(RestApiConstants.DEFAULT_UNLIMITED_LIFETIME);
}
return;
}
}
// If artifact is of type APPLICATION, add a slider specific property
if (application.getArtifact().getType()
== Artifact.TypeEnum.APPLICATION) {
if (application.getConfiguration() == null) {
application.setConfiguration(new Configuration());
}
// Validate there are no component name collisions (collisions are not
// currently supported) and add any components from external applications
// TODO allow name collisions? see AppState#roles
// TODO or add prefix to external component names?
Configuration globalConf = application.getConfiguration();
Set<String> componentNames = new HashSet<>();
List<Component> componentsToRemove = new ArrayList<>();
List<Component> componentsToAdd = new ArrayList<>();
for (Component comp : application.getComponents()) {
if (componentNames.contains(comp.getName())) {
throw new IllegalArgumentException("Component name collision: " +
comp.getName());
}
// resource
validateApplicationResource(application.getResource(), null,
application.getArtifact().getType());
// container size
if (application.getNumberOfContainers() == null
|| application.getNumberOfContainers() < 0) {
throw new IllegalArgumentException(
RestApiErrorMessages.ERROR_CONTAINERS_COUNT_INVALID + ": "
+ application.getNumberOfContainers());
}
validateConfigFile(application.getConfiguration().getFiles(), fs);
// Since it is a simple app with no components, create a default component
application.getComponents().add(createDefaultComponent(application));
} else {
// If the application has components, then run checks for each component.
// Let global values take effect if component level values are not
// provided.
Artifact globalArtifact = application.getArtifact();
Resource globalResource = application.getResource();
Long globalNumberOfContainers = application.getNumberOfContainers();
for (Component comp : application.getComponents()) {
// artifact
if (comp.getArtifact() == null) {
comp.setArtifact(globalArtifact);
}
// If still null raise validation exception
if (comp.getArtifact() == null) {
throw new IllegalArgumentException(String
.format(RestApiErrorMessages.ERROR_ARTIFACT_FOR_COMP_INVALID,
comp.getName()));
}
// If artifact is of type APPLICATION (which cannot be filled from
// global), read external application and add its components to this
// application
if (comp.getArtifact() != null && comp.getArtifact().getType() ==
Artifact.TypeEnum.APPLICATION) {
if (StringUtils.isEmpty(comp.getArtifact().getId())) {
throw new IllegalArgumentException(String
.format(RestApiErrorMessages.ERROR_ARTIFACT_ID_FOR_COMP_INVALID,
comp.getName()));
throw new IllegalArgumentException(
RestApiErrorMessages.ERROR_ARTIFACT_ID_INVALID);
}
// If artifact is of type APPLICATION, add a slider specific property
if (comp.getArtifact().getType() == Artifact.TypeEnum.APPLICATION) {
if (comp.getConfiguration() == null) {
comp.setConfiguration(new Configuration());
LOG.info("Marking {} for removal", comp.getName());
componentsToRemove.add(comp);
List<Component> externalComponents = getApplicationComponents(fs,
comp.getArtifact().getId());
for (Component c : externalComponents) {
Component override = application.getComponent(c.getName());
if (override != null && override.getArtifact() == null) {
// allow properties from external components to be overridden /
// augmented by properties in this component, except for artifact
// which must be read from external component
override.mergeFrom(c);
LOG.info("Merging external component {} from external {}", c
.getName(), comp.getName());
} else {
if (componentNames.contains(c.getName())) {
throw new IllegalArgumentException("Component name collision: " +
c.getName());
}
componentNames.add(c.getName());
componentsToAdd.add(c);
LOG.info("Adding component {} from external {}", c.getName(),
comp.getName());
}
comp.setName(comp.getArtifact().getId());
}
// resource
if (comp.getResource() == null) {
comp.setResource(globalResource);
}
validateApplicationResource(comp.getResource(), comp,
comp.getArtifact().getType());
// container count
if (comp.getNumberOfContainers() == null) {
comp.setNumberOfContainers(globalNumberOfContainers);
}
if (comp.getNumberOfContainers() == null
|| comp.getNumberOfContainers() < 0) {
throw new IllegalArgumentException(String.format(
RestApiErrorMessages.ERROR_CONTAINERS_COUNT_FOR_COMP_INVALID
+ ": " + comp.getNumberOfContainers(), comp.getName()));
}
validateConfigFile(comp.getConfiguration().getFiles(), fs);
} else {
// otherwise handle as a normal component
componentNames.add(comp.getName());
// configuration
comp.getConfiguration().mergeFrom(globalConf);
}
}
application.getComponents().removeAll(componentsToRemove);
application.getComponents().addAll(componentsToAdd);
// Validate components and let global values take effect if component level
// values are not provided
Artifact globalArtifact = application.getArtifact();
Resource globalResource = application.getResource();
Long globalNumberOfContainers = application.getNumberOfContainers();
String globalLaunchCommand = application.getLaunchCommand();
for (Component comp : application.getComponents()) {
// fill in global artifact unless it is type APPLICATION
if (comp.getArtifact() == null && application.getArtifact() != null
&& application.getArtifact().getType() != Artifact.TypeEnum
.APPLICATION) {
comp.setArtifact(globalArtifact);
}
// fill in global resource
if (comp.getResource() == null) {
comp.setResource(globalResource);
}
// fill in global container count
if (comp.getNumberOfContainers() == null) {
comp.setNumberOfContainers(globalNumberOfContainers);
}
// fill in global launch command
if (comp.getLaunchCommand() == null) {
comp.setLaunchCommand(globalLaunchCommand);
}
validateComponent(comp, fs.getFileSystem());
}
// Application lifetime if not specified, is set to unlimited lifetime
if (application.getLifetime() == null) {
@ -145,52 +185,54 @@ public class ServiceApiUtil {
}
}
// 1) Verify the src_file exists and non-empty for template
// 2) dest_file is absolute path
private static void validateConfigFile(List<ConfigFile> list, FileSystem fs)
public static void validateComponent(Component comp, FileSystem fs)
throws IOException {
Set<String> destFileSet = new HashSet<>();
AbstractClientProvider compClientProvider = SliderProviderFactory
.getClientProvider(comp.getArtifact());
compClientProvider.validateArtifact(comp.getArtifact(), fs);
for (ConfigFile file : list) {
if (file.getType().equals(ConfigFile.TypeEnum.TEMPLATE) && StringUtils
.isEmpty(file.getSrcFile())) {
throw new IllegalArgumentException(
"Src_file is empty for " + ConfigFile.TypeEnum.TEMPLATE);
}
if (!StringUtils.isEmpty(file.getSrcFile())) {
Path p = new Path(file.getSrcFile());
if (!fs.exists(p)) {
throw new IllegalArgumentException(
"Src_file does not exist for config file: " + file
.getSrcFile());
}
}
if (StringUtils.isEmpty(file.getDestFile())) {
throw new IllegalArgumentException("Dest_file is empty.");
}
// validate dest_file is absolute
if (!Paths.get(file.getDestFile()).isAbsolute()) {
throw new IllegalArgumentException(
"Dest_file must be absolute path: " + file.getDestFile());
}
if (destFileSet.contains(file.getDestFile())) {
throw new IllegalArgumentException(
"Duplicated ConfigFile exists: " + file.getDestFile());
}
destFileSet.add(file.getDestFile());
if (comp.getLaunchCommand() == null && (comp.getArtifact() == null || comp
.getArtifact().getType() != Artifact.TypeEnum.DOCKER)) {
throw new IllegalArgumentException(RestApiErrorMessages
.ERROR_ABSENT_LAUNCH_COMMAND);
}
validateApplicationResource(comp.getResource(), comp);
if (comp.getNumberOfContainers() == null
|| comp.getNumberOfContainers() < 0) {
throw new IllegalArgumentException(String.format(
RestApiErrorMessages.ERROR_CONTAINERS_COUNT_FOR_COMP_INVALID
+ ": " + comp.getNumberOfContainers(), comp.getName()));
}
compClientProvider.validateConfigFiles(comp.getConfiguration()
.getFiles(), fs);
}
@VisibleForTesting
public static List<Component> getApplicationComponents(SliderFileSystem
fs, String appName) throws IOException {
return loadApplication(fs, appName).getComponents();
}
public static Application loadApplication(SliderFileSystem fs, String
appName) throws IOException {
Path appJson = getAppJsonPath(fs, appName);
LOG.info("Loading application definition from " + appJson);
Application externalApplication = jsonSerDeser.load(fs.getFileSystem(),
appJson);
return externalApplication;
}
public static Path getAppJsonPath(SliderFileSystem fs, String appName) {
Path appDir = fs.buildClusterDirPath(appName);
Path appJson = new Path(appDir, appName + ".json");
return appJson;
}
private static void validateApplicationResource(Resource resource,
Component comp, Artifact.TypeEnum artifactType) {
Component comp) {
// Only apps/components of type APPLICATION can skip resource requirement
if (resource == null && artifactType == Artifact.TypeEnum.APPLICATION) {
return;
}
if (resource == null) {
throw new IllegalArgumentException(
comp == null ? RestApiErrorMessages.ERROR_RESOURCE_INVALID : String
@ -255,6 +297,7 @@ public class ServiceApiUtil {
comp.setResource(app.getResource());
comp.setNumberOfContainers(app.getNumberOfContainers());
comp.setLaunchCommand(app.getLaunchCommand());
comp.setConfiguration(app.getConfiguration());
return comp;
}

View File

@ -34,6 +34,7 @@ import org.apache.slider.core.exceptions.BadCommandArgumentsException;
import org.apache.slider.core.exceptions.SliderException;
import org.apache.slider.core.main.ServiceLauncher;
import org.apache.slider.utils.SliderTestBase;
import org.junit.After;
import org.junit.Before;
import org.junit.Test;
@ -50,6 +51,7 @@ import java.util.UUID;
public class TestKeytabCommandOptions extends SliderTestBase {
private static SliderFileSystem testFileSystem;
private File testFolderDir;
@Before
public void setupFilesystem() throws IOException {
@ -57,11 +59,18 @@ public class TestKeytabCommandOptions extends SliderTestBase {
YarnConfiguration configuration = SliderUtils.createConfiguration();
fileSystem.setConf(configuration);
testFileSystem = new SliderFileSystem(fileSystem, configuration);
File testFolderDir = new File(testFileSystem
testFolderDir = new File(testFileSystem
.buildKeytabInstallationDirPath("").toUri().getPath());
FileUtils.deleteDirectory(testFolderDir);
}
@After
public void cleanup() throws IOException {
if (testFolderDir != null && testFolderDir.exists()) {
FileUtils.deleteDirectory(testFolderDir);
}
}
@Test
public void testInstallKeytab() throws Throwable {
// create a mock keytab file

View File

@ -1,49 +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.slider.common.tools;
import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.slider.utils.SliderTestBase;
import org.junit.Test;
import java.net.URI;
/**
* Test slider utils.
*/
public class TestMiscSliderUtils extends SliderTestBase {
public static final String CLUSTER1 = "cluster1";
@Test
public void testPurgeTempDir() throws Throwable {
Configuration configuration = new Configuration();
FileSystem fs = FileSystem.get(new URI("file:///"), configuration);
SliderFileSystem sliderFileSystem = new SliderFileSystem(fs, configuration);
Path inst = sliderFileSystem.createAppInstanceTempPath(CLUSTER1, "001");
assertTrue(fs.exists(inst));
sliderFileSystem.purgeAppInstanceTempFiles(CLUSTER1);
assertFalse(fs.exists(inst));
}
}

View File

@ -29,18 +29,20 @@ import static org.apache.slider.utils.SliderTestUtils.JSON_SER_DESER;
/**
* Names of the example configs.
*/
public final class ExampleConfResources {
public final class ExampleAppJson {
public static final String APP_JSON = "app.json";
public static final String APP_RES = "app-resolved.json";
public static final String OVERRIDE_JSON = "app-override.json";
public static final String OVERRIDE_RES = "app-override-resolved.json";
public static final String DEFAULT_JSON = "default.json";
public static final String EXTERNAL_JSON_0 = "external0.json";
public static final String EXTERNAL_JSON_1 = "external1.json";
public static final String EXTERNAL_JSON_2 = "external2.json";
public static final String PACKAGE = "/org/apache/slider/core/conf/examples/";
private static final String[] ALL_EXAMPLES = {APP_JSON, APP_RES,
OVERRIDE_JSON, OVERRIDE_RES};
private static final String[] ALL_EXAMPLES = {APP_JSON, OVERRIDE_JSON,
DEFAULT_JSON};
public static final List<String> ALL_EXAMPLE_RESOURCES = new ArrayList<>();
static {
@ -49,10 +51,14 @@ public final class ExampleConfResources {
}
}
private ExampleConfResources() {
private ExampleAppJson() {
}
static Application loadResource(String name) throws IOException {
return JSON_SER_DESER.fromResource(PACKAGE + name);
}
public static String resourceName(String name) {
return "target/test-classes" + PACKAGE + name;
}
}

View File

@ -18,20 +18,40 @@
package org.apache.slider.core.conf;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.slider.api.resource.Application;
import org.apache.slider.api.resource.ConfigFile;
import org.apache.slider.api.resource.ConfigFile.TypeEnum;
import org.apache.slider.api.resource.Configuration;
import org.apache.slider.common.tools.SliderFileSystem;
import org.apache.slider.common.tools.SliderUtils;
import org.apache.slider.core.persist.JsonSerDeser;
import org.apache.slider.util.ServiceApiUtil;
import org.junit.Assert;
import org.junit.Test;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import java.io.IOException;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Map;
import java.util.Set;
import static org.apache.slider.api.InternalKeys.CHAOS_MONKEY_INTERVAL;
import static org.apache.slider.api.InternalKeys.DEFAULT_CHAOS_MONKEY_INTERVAL_DAYS;
import static org.apache.slider.api.InternalKeys.DEFAULT_CHAOS_MONKEY_INTERVAL_HOURS;
import static org.apache.slider.api.InternalKeys.DEFAULT_CHAOS_MONKEY_INTERVAL_MINUTES;
import static org.apache.slider.core.conf.ExampleConfResources.APP_JSON;
import static org.apache.slider.core.conf.ExampleConfResources.OVERRIDE_JSON;
import static org.apache.slider.core.conf.ExampleAppJson.APP_JSON;
import static org.apache.slider.core.conf.ExampleAppJson.EXTERNAL_JSON_1;
import static org.apache.slider.core.conf.ExampleAppJson.OVERRIDE_JSON;
import static org.easymock.EasyMock.anyObject;
import static org.easymock.EasyMock.createNiceMock;
import static org.easymock.EasyMock.expect;
import static org.easymock.EasyMock.replay;
import static org.easymock.EasyMock.reset;
/**
* Test global configuration resolution.
@ -42,23 +62,26 @@ public class TestConfigurationResolve extends Assert {
@Test
public void testOverride() throws Throwable {
Application orig = ExampleConfResources.loadResource(OVERRIDE_JSON);
Application orig = ExampleAppJson.loadResource(OVERRIDE_JSON);
Configuration global = orig.getConfiguration();
assertEquals("a", global.getProperty("g1"));
assertEquals("b", global.getProperty("g2"));
assertEquals(2, global.getFiles().size());
Configuration simple = orig.getComponent("simple").getConfiguration();
assertEquals(0, simple.getProperties().size());
assertEquals(1, simple.getFiles().size());
Configuration master = orig.getComponent("master").getConfiguration();
assertEquals("m", master.getProperty("name"));
assertEquals("overridden", master.getProperty("g1"));
assertEquals(0, master.getFiles().size());
Configuration worker = orig.getComponent("worker").getConfiguration();
LOG.info("worker = {}", worker);
assertEquals(3, worker.getProperties().size());
assertEquals(0, worker.getFiles().size());
assertEquals("worker", worker.getProperty("name"));
assertEquals("overridden-by-worker", worker.getProperty("g1"));
@ -66,18 +89,36 @@ public class TestConfigurationResolve extends Assert {
assertEquals("1000", worker.getProperty("timeout"));
// here is the resolution
SliderUtils.resolve(orig);
SliderFileSystem sfs = createNiceMock(SliderFileSystem.class);
FileSystem mockFs = createNiceMock(FileSystem.class);
expect(sfs.getFileSystem()).andReturn(mockFs).anyTimes();
expect(sfs.buildClusterDirPath(anyObject())).andReturn(
new Path("cluster_dir_path")).anyTimes();
replay(sfs, mockFs);
ServiceApiUtil.validateAndResolveApplication(orig, sfs);
global = orig.getConfiguration();
LOG.info("global = {}", global);
assertEquals("a", global.getProperty("g1"));
assertEquals("b", global.getProperty("g2"));
assertEquals(2, global.getFiles().size());
simple = orig.getComponent("simple").getConfiguration();
assertEquals(2, simple.getProperties().size());
assertEquals("a", simple.getProperty("g1"));
assertEquals("b", simple.getProperty("g2"));
assertEquals(2, simple.getFiles().size());
Set<ConfigFile> files = new HashSet<>();
Map<String, String> props = new HashMap<>();
props.put("k1", "overridden");
props.put("k2", "v2");
files.add(new ConfigFile().destFile("file1").type(TypeEnum
.PROPERTIES).props(props));
files.add(new ConfigFile().destFile("file2").type(TypeEnum
.XML).props(Collections.singletonMap("k3", "v3")));
assertTrue(files.contains(simple.getFiles().get(0)));
assertTrue(files.contains(simple.getFiles().get(1)));
master = orig.getComponent("master").getConfiguration();
LOG.info("master = {}", master);
@ -85,6 +126,17 @@ public class TestConfigurationResolve extends Assert {
assertEquals("m", master.getProperty("name"));
assertEquals("overridden", master.getProperty("g1"));
assertEquals("b", master.getProperty("g2"));
assertEquals(2, master.getFiles().size());
props.put("k1", "v1");
files.clear();
files.add(new ConfigFile().destFile("file1").type(TypeEnum
.PROPERTIES).props(props));
files.add(new ConfigFile().destFile("file2").type(TypeEnum
.XML).props(Collections.singletonMap("k3", "v3")));
assertTrue(files.contains(master.getFiles().get(0)));
assertTrue(files.contains(master.getFiles().get(1)));
worker = orig.getComponent("worker").getConfiguration();
LOG.info("worker = {}", worker);
@ -94,13 +146,91 @@ public class TestConfigurationResolve extends Assert {
assertEquals("overridden-by-worker", worker.getProperty("g1"));
assertEquals("b", worker.getProperty("g2"));
assertEquals("1000", worker.getProperty("timeout"));
assertEquals(2, worker.getFiles().size());
assertTrue(files.contains(worker.getFiles().get(0)));
assertTrue(files.contains(worker.getFiles().get(1)));
}
@Test
public void testOverrideExternalConfiguration() throws IOException {
Application orig = ExampleAppJson.loadResource(EXTERNAL_JSON_1);
Configuration global = orig.getConfiguration();
assertEquals(0, global.getProperties().size());
assertEquals(3, orig.getComponents().size());
Configuration simple = orig.getComponent("simple").getConfiguration();
assertEquals(0, simple.getProperties().size());
Configuration master = orig.getComponent("master").getConfiguration();
assertEquals(1, master.getProperties().size());
assertEquals("is-overridden", master.getProperty("g3"));
Configuration other = orig.getComponent("other").getConfiguration();
assertEquals(0, other.getProperties().size());
// load the external application
SliderFileSystem sfs = createNiceMock(SliderFileSystem.class);
FileSystem mockFs = createNiceMock(FileSystem.class);
expect(sfs.getFileSystem()).andReturn(mockFs).anyTimes();
expect(sfs.buildClusterDirPath(anyObject())).andReturn(
new Path("cluster_dir_path")).anyTimes();
replay(sfs, mockFs);
Application ext = ExampleAppJson.loadResource(APP_JSON);
ServiceApiUtil.validateAndResolveApplication(ext, sfs);
reset(sfs, mockFs);
// perform the resolution on original application
JsonSerDeser<Application> jsonSerDeser = createNiceMock(JsonSerDeser
.class);
expect(sfs.getFileSystem()).andReturn(mockFs).anyTimes();
expect(sfs.buildClusterDirPath(anyObject())).andReturn(
new Path("cluster_dir_path")).anyTimes();
expect(jsonSerDeser.load(anyObject(), anyObject())).andReturn(ext)
.anyTimes();
replay(sfs, mockFs, jsonSerDeser);
ServiceApiUtil.setJsonSerDeser(jsonSerDeser);
ServiceApiUtil.validateAndResolveApplication(orig, sfs);
global = orig.getConfiguration();
assertEquals(0, global.getProperties().size());
assertEquals(4, orig.getComponents().size());
simple = orig.getComponent("simple").getConfiguration();
assertEquals(3, simple.getProperties().size());
assertEquals("a", simple.getProperty("g1"));
assertEquals("b", simple.getProperty("g2"));
assertEquals("60",
simple.getProperty("internal.chaos.monkey.interval.seconds"));
master = orig.getComponent("master").getConfiguration();
assertEquals(5, master.getProperties().size());
assertEquals("512M", master.getProperty("jvm.heapsize"));
assertEquals("overridden", master.getProperty("g1"));
assertEquals("b", master.getProperty("g2"));
assertEquals("is-overridden", master.getProperty("g3"));
assertEquals("60",
simple.getProperty("internal.chaos.monkey.interval.seconds"));
Configuration worker = orig.getComponent("worker").getConfiguration();
LOG.info("worker = {}", worker);
assertEquals(4, worker.getProperties().size());
assertEquals("512M", worker.getProperty("jvm.heapsize"));
assertEquals("overridden-by-worker", worker.getProperty("g1"));
assertEquals("b", worker.getProperty("g2"));
assertEquals("60",
worker.getProperty("internal.chaos.monkey.interval.seconds"));
other = orig.getComponent("other").getConfiguration();
assertEquals(0, other.getProperties().size());
}
@Test
public void testTimeIntervalLoading() throws Throwable {
Application orig = ExampleConfResources.loadResource(APP_JSON);
Application orig = ExampleAppJson.loadResource(APP_JSON);
Configuration conf = orig.getConfiguration();
long s = conf.getPropertyLong(

View File

@ -18,8 +18,11 @@
package org.apache.slider.core.conf;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.slider.api.resource.Application;
import org.apache.slider.common.tools.SliderUtils;
import org.apache.slider.common.tools.SliderFileSystem;
import org.apache.slider.util.ServiceApiUtil;
import org.junit.Assert;
import org.junit.Test;
import org.junit.runner.RunWith;
@ -29,24 +32,28 @@ import java.util.Arrays;
import java.util.Collection;
import static org.apache.slider.utils.SliderTestUtils.JSON_SER_DESER;
import static org.easymock.EasyMock.anyObject;
import static org.easymock.EasyMock.createNiceMock;
import static org.easymock.EasyMock.expect;
import static org.easymock.EasyMock.replay;
/**
* Test loading example resources.
*/
@RunWith(value = Parameterized.class)
public class TestConfTreeLoadExamples extends Assert {
public class TestExampleAppJson extends Assert {
private String resource;
public TestConfTreeLoadExamples(String resource) {
public TestExampleAppJson(String resource) {
this.resource = resource;
}
@Parameterized.Parameters
public static Collection<String[]> filenames() {
String[][] stringArray = new String[ExampleConfResources
String[][] stringArray = new String[ExampleAppJson
.ALL_EXAMPLE_RESOURCES.size()][1];
int i = 0;
for (String s : ExampleConfResources.ALL_EXAMPLE_RESOURCES) {
for (String s : ExampleAppJson.ALL_EXAMPLE_RESOURCES) {
stringArray[i++][0] = s;
}
return Arrays.asList(stringArray);
@ -56,7 +63,15 @@ public class TestConfTreeLoadExamples extends Assert {
public void testLoadResource() throws Throwable {
try {
Application application = JSON_SER_DESER.fromResource(resource);
SliderUtils.resolve(application);
SliderFileSystem sfs = createNiceMock(SliderFileSystem.class);
FileSystem mockFs = createNiceMock(FileSystem.class);
expect(sfs.getFileSystem()).andReturn(mockFs).anyTimes();
expect(sfs.buildClusterDirPath(anyObject())).andReturn(
new Path("cluster_dir_path")).anyTimes();
replay(sfs, mockFs);
ServiceApiUtil.validateAndResolveApplication(application, sfs);
} catch (Exception e) {
throw new Exception("exception loading " + resource + ":" + e.toString());
}

View File

@ -0,0 +1,121 @@
/*
* 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.slider.providers;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.slider.api.resource.Artifact;
import org.apache.slider.api.resource.ConfigFile;
import org.apache.slider.api.resource.ConfigFile.TypeEnum;
import org.junit.Assert;
import org.junit.Test;
import java.io.IOException;
import java.util.ArrayList;
import java.util.List;
import static org.easymock.EasyMock.anyObject;
import static org.easymock.EasyMock.createNiceMock;
import static org.easymock.EasyMock.expect;
import static org.easymock.EasyMock.replay;
/**
* Test the AbstractClientProvider shared methods.
*/
public class TestAbstractClientProvider {
private static final String EXCEPTION_PREFIX = "Should have thrown " +
"exception: ";
private static final String NO_EXCEPTION_PREFIX = "Should not have thrown " +
"exception: ";
private static class ClientProvider extends AbstractClientProvider {
@Override
public void validateArtifact(Artifact artifact, FileSystem fileSystem)
throws IOException {
}
@Override
protected void validateConfigFile(ConfigFile configFile,
FileSystem fileSystem) throws IOException {
}
}
@Test
public void testConfigFiles() throws IOException {
ClientProvider clientProvider = new ClientProvider();
FileSystem mockFs = createNiceMock(FileSystem.class);
expect(mockFs.exists(anyObject(Path.class))).andReturn(true).anyTimes();
replay(mockFs);
ConfigFile configFile = new ConfigFile();
List<ConfigFile> configFiles = new ArrayList<>();
configFiles.add(configFile);
try {
clientProvider.validateConfigFiles(configFiles, mockFs);
Assert.fail(EXCEPTION_PREFIX + "null file type");
} catch (IllegalArgumentException e) {
}
configFile.setType(TypeEnum.TEMPLATE);
try {
clientProvider.validateConfigFiles(configFiles, mockFs);
Assert.fail(EXCEPTION_PREFIX + "empty src_file for type template");
} catch (IllegalArgumentException e) {
}
configFile.setSrcFile("srcfile");
try {
clientProvider.validateConfigFiles(configFiles, mockFs);
Assert.fail(EXCEPTION_PREFIX + "empty dest file");
} catch (IllegalArgumentException e) {
}
configFile.setDestFile("destfile");
try {
clientProvider.validateConfigFiles(configFiles, mockFs);
} catch (IllegalArgumentException e) {
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
configFile = new ConfigFile();
configFile.setType(TypeEnum.JSON);
configFile.setSrcFile(null);
configFile.setDestFile("path/destfile2");
configFiles.add(configFile);
try {
clientProvider.validateConfigFiles(configFiles, mockFs);
Assert.fail(EXCEPTION_PREFIX + "dest file with multiple path elements");
} catch (IllegalArgumentException e) {
}
configFile.setDestFile("/path/destfile2");
try {
clientProvider.validateConfigFiles(configFiles, mockFs);
} catch (IllegalArgumentException e) {
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
configFile.setDestFile("destfile");
try {
clientProvider.validateConfigFiles(configFiles, mockFs);
Assert.fail(EXCEPTION_PREFIX + "duplicate dest file");
} catch (IllegalArgumentException e) {
}
}
}

View File

@ -0,0 +1,96 @@
/*
* 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.slider.providers;
import org.apache.slider.api.resource.Component;
import org.apache.slider.client.SliderClient;
import org.apache.slider.common.params.SliderActions;
import org.apache.slider.common.tools.SliderFileSystem;
import org.apache.slider.core.conf.ExampleAppJson;
import org.apache.slider.core.main.ServiceLauncher;
import org.apache.slider.util.ServiceApiUtil;
import org.apache.slider.utils.YarnZKMiniClusterTestBase;
import org.junit.Test;
import java.util.Arrays;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
import static org.apache.slider.common.params.Arguments.ARG_APPDEF;
/**
* Test for building / resolving components of type APPLICATION.
*/
public class TestBuildApplicationComponent extends YarnZKMiniClusterTestBase {
private static void checkComponentNames(List<Component> components,
Set<String> names) {
assertEquals(names.size(), components.size());
for (Component comp : components) {
assertTrue(names.contains(comp.getName()));
}
}
public void buildAndCheckComponents(String appName, String appDef,
SliderFileSystem sfs, Set<String> names) throws Throwable {
ServiceLauncher<SliderClient> launcher = createOrBuildCluster(
SliderActions.ACTION_BUILD, appName, Arrays.asList(ARG_APPDEF,
ExampleAppJson.resourceName(appDef)), true, false);
SliderClient sliderClient = launcher.getService();
addToTeardown(sliderClient);
// verify the cluster exists
assertEquals(0, sliderClient.actionExists(appName, false));
// verify generated conf
List<Component> components = ServiceApiUtil.getApplicationComponents(sfs,
appName);
checkComponentNames(components, names);
}
@Test
public void testExternalComponentBuild() throws Throwable {
String clustername = createMiniCluster("", getConfiguration(), 1, true);
describe("verify external components");
SliderFileSystem sfs = createSliderFileSystem();
Set<String> nameSet = new HashSet<>();
nameSet.add("simple");
nameSet.add("master");
nameSet.add("worker");
buildAndCheckComponents("app-1", ExampleAppJson.APP_JSON, sfs,
nameSet);
buildAndCheckComponents("external-0", ExampleAppJson
.EXTERNAL_JSON_0, sfs, nameSet);
nameSet.add("other");
buildAndCheckComponents("external-1", ExampleAppJson
.EXTERNAL_JSON_1, sfs, nameSet);
nameSet.add("another");
buildAndCheckComponents("external-2", ExampleAppJson
.EXTERNAL_JSON_2, sfs, nameSet);
}
}

View File

@ -0,0 +1,60 @@
/*
* 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.slider.providers;
import org.apache.slider.api.resource.Application;
import org.apache.slider.client.SliderClient;
import org.apache.slider.common.params.SliderActions;
import org.apache.slider.core.conf.ExampleAppJson;
import org.apache.slider.core.main.ServiceLauncher;
import org.apache.slider.utils.YarnZKMiniClusterTestBase;
import org.junit.Ignore;
import org.junit.Test;
import java.util.Arrays;
import static org.apache.slider.common.params.Arguments.ARG_APPDEF;
/**
* Simple end-to-end test.
*/
public class TestDefaultProvider extends YarnZKMiniClusterTestBase {
// TODO figure out how to run client commands against minicluster
// (currently errors out unable to find containing jar of AM for upload)
@Ignore
@Test
public void testDefaultProvider() throws Throwable {
createMiniCluster("", getConfiguration(), 1, true);
String appName = "default-1";
describe("verify default provider");
String appDef = ExampleAppJson.resourceName(ExampleAppJson
.DEFAULT_JSON);
ServiceLauncher<SliderClient> launcher = createOrBuildCluster(
SliderActions.ACTION_CREATE, appName, Arrays.asList(ARG_APPDEF,
appDef), true, true);
SliderClient sliderClient = launcher.getService();
addToTeardown(sliderClient);
Application application = sliderClient.actionStatus(appName);
assertEquals(1L, application.getContainers().size());
}
}

View File

@ -43,7 +43,7 @@ public class BaseMockAppStateAATest extends BaseMockAppStateTest
@Override
public Application buildApplication() {
Application application = factory.newApplication(0, 0, 0)
.name(getTestName());
.name(getValidTestName());
application.getComponent(ROLE1).getConfiguration().setProperty(
COMPONENT_PLACEMENT_POLICY, Integer.toString(PlacementPolicy
.ANTI_AFFINITY_REQUIRED));

View File

@ -362,7 +362,7 @@ public class TestMockAppStateAAPlacement extends BaseMockAppStateAATest
// now destroy the app state
AppStateBindingInfo bindingInfo = buildBindingInfo();
bindingInfo.application = factory.newApplication(0, 0, desiredAA).name(
getTestName());
getValidTestName());
bindingInfo.application.getComponent(ROLE2)
.getConfiguration().setProperty(COMPONENT_PLACEMENT_POLICY,
Integer.toString(PlacementPolicy.ANTI_AFFINITY_REQUIRED));

View File

@ -203,7 +203,7 @@ public class TestMockAppStateContainerFailure extends BaseMockAppStateTest
// Update instance definition to allow containers to fail any number of
// times
AppStateBindingInfo bindingInfo = buildBindingInfo();
bindingInfo.application.getConfiguration().setProperty(
bindingInfo.application.getComponent(ROLE0).getConfiguration().setProperty(
ResourceKeys.CONTAINER_FAILURE_THRESHOLD, "0");
appState = new MockAppState(bindingInfo);

View File

@ -38,6 +38,7 @@ import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import java.io.File;
import java.io.IOException;
import java.util.Collections;
/**
@ -65,7 +66,7 @@ public class TestMockAppStateFlexDynamicRoles extends BaseMockAppStateTest
}
@Override
public AppStateBindingInfo buildBindingInfo() {
public AppStateBindingInfo buildBindingInfo() throws IOException {
AppStateBindingInfo bindingInfo = super.buildBindingInfo();
bindingInfo.releaseSelector = new MostRecentContainerReleaseSelector();
return bindingInfo;
@ -145,7 +146,7 @@ public class TestMockAppStateFlexDynamicRoles extends BaseMockAppStateTest
appState = new MockAppState();
AppStateBindingInfo binding2 = buildBindingInfo();
binding2.application = factory.newApplication(0, 0, 0)
.name(getTestName());
.name(getValidTestName());
binding2.historyPath = historyPath2;
appState.buildInstance(binding2);
// on this read there won't be the right number of roles

View File

@ -70,7 +70,7 @@ public class TestMockAppStateRebuildOnAMRestart extends BaseMockAppStateTest
AppStateBindingInfo bindingInfo = buildBindingInfo();
bindingInfo.application = factory.newApplication(r0, r1, r2)
.name(getTestName());
.name(getValidTestName());
bindingInfo.liveContainers = containers;
appState = new MockAppState(bindingInfo);

View File

@ -30,6 +30,7 @@ import org.apache.slider.server.appmaster.state.RoleInstance;
import org.apache.slider.server.appmaster.state.RoleStatus;
import org.junit.Test;
import java.io.IOException;
import java.util.Collections;
import java.util.List;
import java.util.Map;
@ -59,7 +60,7 @@ public class TestMockAppStateUniqueNames extends BaseMockAppStateTest
}
@Override
public AppStateBindingInfo buildBindingInfo() {
public AppStateBindingInfo buildBindingInfo() throws IOException {
AppStateBindingInfo bindingInfo = super.buildBindingInfo();
bindingInfo.releaseSelector = new MostRecentContainerReleaseSelector();
return bindingInfo;

View File

@ -40,7 +40,7 @@ public class TestMockContainerResourceAllocations extends BaseMockAppStateTest {
@Override
public Application buildApplication() {
return factory.newApplication(1, 0, 0).name(getTestName());
return factory.newApplication(1, 0, 0).name(getValidTestName());
}
@Test

View File

@ -51,6 +51,7 @@ import org.apache.slider.server.appmaster.state.ProviderAppState;
import org.apache.slider.server.appmaster.state.RoleInstance;
import org.apache.slider.server.appmaster.state.RoleStatus;
import org.apache.slider.server.appmaster.state.StateAccessForProviders;
import org.apache.slider.util.ServiceApiUtil;
import org.apache.slider.utils.SliderTestBase;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@ -62,6 +63,7 @@ import java.net.URI;
import java.util.ArrayList;
import java.util.Collections;
import java.util.List;
import java.util.Locale;
import java.util.Map.Entry;
/**
@ -118,7 +120,7 @@ public abstract class BaseMockAppStateTest extends SliderTestBase implements
historyPath = new Path(historyWorkDir.toURI());
fs.delete(historyPath, true);
appState = new MockAppState(buildBindingInfo());
stateAccess = new ProviderAppState(getTestName(), appState);
stateAccess = new ProviderAppState(getValidTestName(), appState);
}
/**
@ -127,9 +129,11 @@ public abstract class BaseMockAppStateTest extends SliderTestBase implements
* from {@link #buildApplication()} ()}
* @return
*/
protected AppStateBindingInfo buildBindingInfo() {
protected AppStateBindingInfo buildBindingInfo() throws IOException {
AppStateBindingInfo binding = new AppStateBindingInfo();
binding.application = buildApplication();
ServiceApiUtil.validateAndResolveApplication(binding.application,
sliderFileSystem);
//binding.roles = new ArrayList<>(factory.ROLES);
binding.fs = fs;
binding.historyPath = historyPath;
@ -142,7 +146,7 @@ public abstract class BaseMockAppStateTest extends SliderTestBase implements
* @return the instance definition
*/
public Application buildApplication() {
return factory.newApplication(0, 0, 0).name(getTestName());
return factory.newApplication(0, 0, 0).name(getValidTestName());
}
/**
@ -153,6 +157,10 @@ public abstract class BaseMockAppStateTest extends SliderTestBase implements
return methodName.getMethodName();
}
public String getValidTestName() {
return getTestName().toLowerCase(Locale.ENGLISH);
}
public RoleStatus getRole0Status() {
return lookupRole(ROLE0);
}

View File

@ -32,6 +32,7 @@ import org.apache.hadoop.yarn.client.api.AMRMClient;
import org.apache.slider.api.ResourceKeys;
import org.apache.slider.api.resource.Application;
import org.apache.slider.api.resource.Component;
import org.apache.slider.api.resource.Resource;
import org.apache.slider.providers.PlacementPolicy;
import org.apache.slider.providers.ProviderRole;
@ -190,6 +191,8 @@ public class MockFactory implements MockRoles {
*/
public Application newApplication(long r1, long r2, long r3) {
Application application = new Application();
application.setLaunchCommand("sleep 60");
application.setResource(new Resource().memory("256"));
application.getConfiguration().setProperty(ResourceKeys
.NODE_FAILURE_THRESHOLD, Integer.toString(NODE_FAILURE_THRESHOLD));
List<Component> components = application.getComponents();

View File

@ -0,0 +1,393 @@
/*
* 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.slider.utils;
import org.apache.hadoop.fs.FileSystem;
import org.apache.hadoop.fs.Path;
import org.apache.slider.api.resource.Application;
import org.apache.slider.api.resource.Artifact;
import org.apache.slider.api.resource.Component;
import org.apache.slider.api.resource.Resource;
import org.apache.slider.common.tools.SliderFileSystem;
import org.apache.slider.core.persist.JsonSerDeser;
import org.apache.slider.util.RestApiConstants;
import org.apache.slider.util.RestApiErrorMessages;
import org.apache.slider.util.ServiceApiUtil;
import org.junit.Assert;
import org.junit.Test;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import java.io.IOException;
import static org.apache.slider.util.RestApiConstants.DEFAULT_COMPONENT_NAME;
import static org.apache.slider.util.RestApiConstants.DEFAULT_UNLIMITED_LIFETIME;
import static org.apache.slider.util.RestApiErrorMessages.*;
import static org.apache.slider.util.RestApiErrorMessages.ERROR_CONTAINERS_COUNT_INVALID;
import static org.apache.slider.util.RestApiErrorMessages.ERROR_RESOURCE_PROFILE_NOT_SUPPORTED_YET;
import static org.easymock.EasyMock.anyObject;
import static org.easymock.EasyMock.createNiceMock;
import static org.easymock.EasyMock.expect;
import static org.easymock.EasyMock.replay;
import static org.junit.Assert.assertEquals;
import static org.junit.Assert.assertNotNull;
/**
* Test for ServiceApiUtil helper methods.
*/
public class TestServiceApiUtil {
private static final Logger LOG = LoggerFactory
.getLogger(TestServiceApiUtil.class);
private static final String EXCEPTION_PREFIX = "Should have thrown " +
"exception: ";
private static final String NO_EXCEPTION_PREFIX = "Should not have thrown " +
"exception: ";
@Test(timeout = 90000)
public void testResourceValidation() throws Exception {
SliderFileSystem sfs = initMock(null);
Application app = new Application();
// no name
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with no name");
} catch (IllegalArgumentException e) {
assertEquals(ERROR_APPLICATION_NAME_INVALID, e.getMessage());
}
// bad format name
String[] badNames = {"4finance", "Finance", "finance@home"};
for (String badName : badNames) {
app.setName(badName);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with bad name " + badName);
} catch (IllegalArgumentException e) {
assertEquals(String.format(
ERROR_APPLICATION_NAME_INVALID_FORMAT, badName), e.getMessage());
}
}
// launch command not specified
app.setName("finance_home");
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with no launch command");
} catch (IllegalArgumentException e) {
assertEquals(RestApiErrorMessages.ERROR_ABSENT_LAUNCH_COMMAND,
e.getMessage());
}
// resource not specified
app.setLaunchCommand("sleep 3600");
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with no resource");
} catch (IllegalArgumentException e) {
assertEquals(String.format(
RestApiErrorMessages.ERROR_RESOURCE_FOR_COMP_INVALID,
RestApiConstants.DEFAULT_COMPONENT_NAME), e.getMessage());
}
// memory not specified
Resource res = new Resource();
app.setResource(res);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with no memory");
} catch (IllegalArgumentException e) {
assertEquals(String.format(
RestApiErrorMessages.ERROR_RESOURCE_MEMORY_FOR_COMP_INVALID,
RestApiConstants.DEFAULT_COMPONENT_NAME), e.getMessage());
}
// invalid no of cpus
res.setMemory("100mb");
res.setCpus(-2);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(
EXCEPTION_PREFIX + "application with invalid no of cpus");
} catch (IllegalArgumentException e) {
assertEquals(String.format(
RestApiErrorMessages.ERROR_RESOURCE_CPUS_FOR_COMP_INVALID_RANGE,
RestApiConstants.DEFAULT_COMPONENT_NAME), e.getMessage());
}
// number of containers not specified
res.setCpus(2);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with no container count");
} catch (IllegalArgumentException e) {
Assert.assertTrue(e.getMessage()
.contains(ERROR_CONTAINERS_COUNT_INVALID));
}
// specifying profile along with cpus/memory raises exception
res.setProfile("hbase_finance_large");
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX
+ "application with resource profile along with cpus/memory");
} catch (IllegalArgumentException e) {
assertEquals(String.format(RestApiErrorMessages
.ERROR_RESOURCE_PROFILE_MULTIPLE_VALUES_FOR_COMP_NOT_SUPPORTED,
RestApiConstants.DEFAULT_COMPONENT_NAME),
e.getMessage());
}
// currently resource profile alone is not supported.
// TODO: remove the next test once resource profile alone is supported.
res.setCpus(null);
res.setMemory(null);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with resource profile only");
} catch (IllegalArgumentException e) {
assertEquals(ERROR_RESOURCE_PROFILE_NOT_SUPPORTED_YET,
e.getMessage());
}
// unset profile here and add cpus/memory back
res.setProfile(null);
res.setCpus(2);
res.setMemory("2gb");
// null number of containers
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "null number of containers");
} catch (IllegalArgumentException e) {
Assert.assertTrue(e.getMessage()
.startsWith(ERROR_CONTAINERS_COUNT_INVALID));
}
// negative number of containers
app.setNumberOfContainers(-1L);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "negative number of containers");
} catch (IllegalArgumentException e) {
Assert.assertTrue(e.getMessage()
.startsWith(ERROR_CONTAINERS_COUNT_INVALID));
}
// everything valid here
app.setNumberOfContainers(5L);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
} catch (IllegalArgumentException e) {
LOG.error("application attributes specified should be valid here", e);
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
}
@Test
public void testArtifacts() throws IOException {
SliderFileSystem sfs = initMock(null);
Application app = new Application();
app.setName("name");
Resource res = new Resource();
app.setResource(res);
res.setMemory("512M");
app.setNumberOfContainers(3L);
// no artifact id fails with default type
Artifact artifact = new Artifact();
app.setArtifact(artifact);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with no artifact id");
} catch (IllegalArgumentException e) {
assertEquals(ERROR_ARTIFACT_ID_INVALID, e.getMessage());
}
// no artifact id fails with APPLICATION type
artifact.setType(Artifact.TypeEnum.APPLICATION);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with no artifact id");
} catch (IllegalArgumentException e) {
assertEquals(ERROR_ARTIFACT_ID_INVALID, e.getMessage());
}
// no artifact id fails with TARBALL type
artifact.setType(Artifact.TypeEnum.TARBALL);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with no artifact id");
} catch (IllegalArgumentException e) {
assertEquals(ERROR_ARTIFACT_ID_INVALID, e.getMessage());
}
// everything valid here
artifact.setType(Artifact.TypeEnum.DOCKER);
artifact.setId("docker.io/centos:centos7");
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
} catch (IllegalArgumentException e) {
LOG.error("application attributes specified should be valid here", e);
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
// defaults assigned
assertEquals(app.getComponents().get(0).getName(),
DEFAULT_COMPONENT_NAME);
assertEquals(app.getLifetime(), DEFAULT_UNLIMITED_LIFETIME);
}
private static Resource createValidResource() {
Resource res = new Resource();
res.setMemory("512M");
return res;
}
private static Component createValidComponent(String compName) {
Component comp = new Component();
comp.setName(compName);
comp.setResource(createValidResource());
comp.setNumberOfContainers(1L);
return comp;
}
private static Application createValidApplication(String compName) {
Application app = new Application();
app.setLaunchCommand("sleep 3600");
app.setName("name");
app.setResource(createValidResource());
app.setNumberOfContainers(1L);
if (compName != null) {
app.addComponent(createValidComponent(compName));
}
return app;
}
private static SliderFileSystem initMock(Application ext) throws IOException {
SliderFileSystem sfs = createNiceMock(SliderFileSystem.class);
FileSystem mockFs = createNiceMock(FileSystem.class);
JsonSerDeser<Application> jsonSerDeser = createNiceMock(JsonSerDeser
.class);
expect(sfs.getFileSystem()).andReturn(mockFs).anyTimes();
expect(sfs.buildClusterDirPath(anyObject())).andReturn(
new Path("cluster_dir_path")).anyTimes();
if (ext != null) {
expect(jsonSerDeser.load(anyObject(), anyObject())).andReturn(ext)
.anyTimes();
}
replay(sfs, mockFs, jsonSerDeser);
ServiceApiUtil.setJsonSerDeser(jsonSerDeser);
return sfs;
}
@Test
public void testExternalApplication() throws IOException {
Application ext = createValidApplication("comp1");
SliderFileSystem sfs = initMock(ext);
Application app = createValidApplication(null);
Artifact artifact = new Artifact();
artifact.setType(Artifact.TypeEnum.APPLICATION);
artifact.setId("id");
app.setArtifact(artifact);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
} catch (IllegalArgumentException e) {
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
assertEquals(1, app.getComponents().size());
assertNotNull(app.getComponent("comp1"));
}
@Test
public void testDuplicateComponents() throws IOException {
SliderFileSystem sfs = initMock(null);
String compName = "comp1";
Application app = createValidApplication(compName);
app.addComponent(createValidComponent(compName));
// duplicate component name fails
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
Assert.fail(EXCEPTION_PREFIX + "application with component collision");
} catch (IllegalArgumentException e) {
assertEquals("Component name collision: " + compName, e.getMessage());
}
}
@Test
public void testExternalDuplicateComponent() throws IOException {
Application ext = createValidApplication("comp1");
SliderFileSystem sfs = initMock(ext);
Application app = createValidApplication("comp1");
Artifact artifact = new Artifact();
artifact.setType(Artifact.TypeEnum.APPLICATION);
artifact.setId("id");
app.getComponent("comp1").setArtifact(artifact);
// duplicate component name okay in the case of APPLICATION component
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
} catch (IllegalArgumentException e) {
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
}
@Test
public void testExternalComponent() throws IOException {
Application ext = createValidApplication("comp1");
SliderFileSystem sfs = initMock(ext);
Application app = createValidApplication("comp2");
Artifact artifact = new Artifact();
artifact.setType(Artifact.TypeEnum.APPLICATION);
artifact.setId("id");
app.setArtifact(artifact);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
} catch (IllegalArgumentException e) {
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
assertEquals(1, app.getComponents().size());
// artifact ID not inherited from global
assertNotNull(app.getComponent("comp2"));
// set APPLICATION artifact id on component
app.getComponent("comp2").setArtifact(artifact);
try {
ServiceApiUtil.validateAndResolveApplication(app, sfs);
} catch (IllegalArgumentException e) {
Assert.fail(NO_EXCEPTION_PREFIX + e.getMessage());
}
assertEquals(1, app.getComponents().size());
// original component replaced by external component
assertNotNull(app.getComponent("comp1"));
}
}

View File

@ -37,6 +37,7 @@ import org.apache.slider.common.SliderExitCodes;
import org.apache.slider.common.SliderXmlConfKeys;
import org.apache.slider.common.params.ActionFreezeArgs;
import org.apache.slider.common.params.Arguments;
import org.apache.slider.common.params.SliderActions;
import org.apache.slider.common.tools.Duration;
import org.apache.slider.common.tools.SliderFileSystem;
import org.apache.slider.common.tools.SliderUtils;
@ -328,11 +329,8 @@ public abstract class YarnMiniClusterTestBase extends SliderTestBase {
*/
public void stopRunningClusters() {
for (SliderClient client : clustersToTeardown) {
try {
maybeStopCluster(client, "", "Teardown at end of test case", true);
} catch (Exception e) {
LOG.warn("While stopping cluster " + e, e);
}
maybeStopCluster(client, client.getDeployedClusterName(),
"Teardown at end of test case", true);
}
}
@ -501,6 +499,62 @@ public abstract class YarnMiniClusterTestBase extends SliderTestBase {
return buildFsDefaultName(hdfsCluster);
}
/**
* Create or build a cluster (the action is set by the first verb).
* @param action operation to invoke: SliderActions.ACTION_CREATE or
* SliderActions.ACTION_BUILD
* @param clustername cluster name
* @param extraArgs list of extra args to add to the creation command
* @param deleteExistingData should the data of any existing cluster
* of this name be deleted
* @param blockUntilRunning block until the AM is running
* @return launcher which will have executed the command.
*/
public ServiceLauncher<SliderClient> createOrBuildCluster(String action,
String clustername, List<String> extraArgs, boolean deleteExistingData,
boolean blockUntilRunning) throws Throwable {
assertNotNull(clustername);
assertNotNull(miniCluster);
// update action should keep existing data
Configuration config = miniCluster.getConfig();
if (deleteExistingData && !SliderActions.ACTION_UPDATE.equals(action)) {
FileSystem dfs = FileSystem.get(new URI(getFsDefaultName()), config);
SliderFileSystem sliderFileSystem = new SliderFileSystem(dfs, config);
Path clusterDir = sliderFileSystem.buildClusterDirPath(clustername);
LOG.info("deleting instance data at {}", clusterDir);
//this is a safety check to stop us doing something stupid like deleting /
assertTrue(clusterDir.toString().contains("/.slider/"));
rigorousDelete(sliderFileSystem, clusterDir, 60000);
}
List<String> argsList = new ArrayList<>();
argsList.addAll(Arrays.asList(
action, clustername,
Arguments.ARG_MANAGER, getRMAddr(),
Arguments.ARG_FILESYSTEM, getFsDefaultName(),
Arguments.ARG_DEBUG));
argsList.addAll(getExtraCLIArgs());
if (extraArgs != null) {
argsList.addAll(extraArgs);
}
ServiceLauncher<SliderClient> launcher = launchClientAgainstMiniMR(
//config includes RM binding info
new YarnConfiguration(config),
//varargs list of command line params
argsList
);
assertEquals(0, launcher.getServiceExitCode());
SliderClient client = launcher.getService();
if (blockUntilRunning) {
client.monitorAppToRunning(new Duration(CLUSTER_GO_LIVE_TIME));
}
return launcher;
}
/**
* Delete with some pauses and backoff; designed to handle slow delete
* operation in windows.
@ -652,28 +706,6 @@ public abstract class YarnMiniClusterTestBase extends SliderTestBase {
return getTestConfiguration().getTrimmed(getApplicationHomeKey());
}
public List<String> getImageCommands() {
if (switchToImageDeploy) {
// its an image that had better be defined
assertNotNull(getArchivePath());
if (!imageIsRemote) {
// its not remote, so assert it exists
File f = new File(getArchivePath());
assertTrue(f.exists());
return Arrays.asList(Arguments.ARG_IMAGE, f.toURI().toString());
} else {
assertNotNull(remoteImageURI);
// if it is remote, then its whatever the archivePath property refers to
return Arrays.asList(Arguments.ARG_IMAGE, remoteImageURI.toString());
}
} else {
assertNotNull(getApplicationHome());
assertTrue(new File(getApplicationHome()).exists());
return Arrays.asList(Arguments.ARG_APP_HOME, getApplicationHome());
}
}
/**
* Get the resource configuration dir in the source tree.
*
@ -746,14 +778,23 @@ public abstract class YarnMiniClusterTestBase extends SliderTestBase {
SliderClient sliderClient,
String clustername,
String message,
boolean force) throws IOException, YarnException {
boolean force) {
if (sliderClient != null) {
if (SliderUtils.isUnset(clustername)) {
clustername = sliderClient.getDeployedClusterName();
}
//only stop a cluster that exists
if (SliderUtils.isSet(clustername)) {
return clusterActionFreeze(sliderClient, clustername, message, force);
try {
clusterActionFreeze(sliderClient, clustername, message, force);
} catch (Exception e) {
LOG.warn("While stopping cluster " + e, e);
}
try {
sliderClient.actionDestroy(clustername);
} catch (Exception e) {
LOG.warn("While destroying cluster " + e, e);
}
}
}
return 0;

View File

@ -22,7 +22,6 @@ import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.io.IOUtils;
import org.apache.hadoop.registry.client.api.RegistryConstants;
import org.apache.hadoop.yarn.conf.YarnConfiguration;
import org.apache.slider.common.tools.SliderUtils;
import org.apache.slider.core.zk.BlockingZKWatcher;
import org.apache.slider.core.zk.ZKIntegration;
import org.slf4j.Logger;
@ -109,9 +108,7 @@ public abstract class YarnZKMiniClusterTestBase extends
int numLogDirs,
boolean startZK,
boolean startHDFS) throws IOException {
if (SliderUtils.isUnset(name)) {
name = methodName.getMethodName();
}
name = buildClustername(name);
createMicroZKCluster("-" + name, conf);
conf.setBoolean(RegistryConstants.KEY_REGISTRY_ENABLED, true);
conf.set(RegistryConstants.KEY_REGISTRY_ZK_QUORUM, getZKBinding());

View File

@ -1,49 +0,0 @@
{
"name": "app-1",
"lifetime": "3600",
"configuration": {
"properties": {
"g1": "a",
"g2": "b"
}
},
"resource": {
"cpus": 1,
"memory": "512"
},
"number_of_containers": 2,
"components": [
{
"name": "simple",
"configuration": {
"properties": {
"g1": "a",
"g2": "b"
}
}
},
{
"name": "master",
"configuration": {
"properties": {
"g1": "overridden",
"g2": "b"
}
}
},
{
"name": "worker",
"resource": {
"cpus": 1,
"memory": "1024"
},
"configuration": {
"properties": {
"g1": "overridden-by-worker",
"g2": "b",
"timeout": "1000"
}
}
}
]
}

View File

@ -1,11 +1,29 @@
{
"name": "app-1",
"lifetime": "3600",
"launch_command": "sleep 3600",
"configuration": {
"properties": {
"g1": "a",
"g2": "b"
}
},
"files": [
{
"type": "PROPERTIES",
"dest_file": "file1",
"props": {
"k1": "v1",
"k2": "v2"
}
},
{
"type": "XML",
"dest_file": "file2",
"props": {
"k3": "v3"
}
}
]
},
"resource": {
"cpus": 1,
@ -14,7 +32,18 @@
"number_of_containers": 2,
"components": [
{
"name": "simple"
"name": "simple",
"configuration": {
"files": [
{
"type": "PROPERTIES",
"dest_file": "file1",
"props": {
"k1": "overridden"
}
}
]
}
},
{
"name": "master",

View File

@ -1,81 +0,0 @@
{
"name": "zk-app-1",
"lifetime": "3600",
"configuration": {
"properties": {
"internal.chaos.monkey.interval.seconds": "60",
"zookeeper.port": "2181",
"zookeeper.path": "/yarnapps_small_cluster",
"zookeeper.hosts": "zoo1,zoo2,zoo3",
"env.MALLOC_ARENA_MAX": "4",
"site.hbase.master.startup.retainassign": "true",
"site.fs.defaultFS": "hdfs://cluster:8020",
"site.fs.default.name": "hdfs://cluster:8020",
"site.hbase.master.info.port": "0",
"site.hbase.regionserver.info.port": "0"
}
},
"resource": {
"cpus": 1,
"memory": "512"
},
"number_of_containers": 2,
"components": [
{
"name": "simple",
"number_of_containers": 2,
"configuration": {
"properties": {
"g1": "a",
"g2": "b"
}
}
},
{
"name": "master",
"number_of_containers": 1,
"resource": {
"cpus": 1,
"memory": "512"
},
"configuration": {
"properties": {
"zookeeper.port": "2181",
"zookeeper.path": "/yarnapps_small_cluster",
"zookeeper.hosts": "zoo1,zoo2,zoo3",
"env.MALLOC_ARENA_MAX": "4",
"site.hbase.master.startup.retainassign": "true",
"site.fs.defaultFS": "hdfs://cluster:8020",
"site.fs.default.name": "hdfs://cluster:8020",
"site.hbase.master.info.port": "0",
"site.hbase.regionserver.info.port": "0",
"jvm.heapsize": "512M"
}
}
},
{
"name": "worker",
"number_of_containers": 5,
"resource": {
"cpus": 1,
"memory": "1024"
},
"configuration": {
"properties": {
"g1": "overridden-by-worker",
"g2": "b",
"zookeeper.port": "2181",
"zookeeper.path": "/yarnapps_small_cluster",
"zookeeper.hosts": "zoo1,zoo2,zoo3",
"env.MALLOC_ARENA_MAX": "4",
"site.hbase.master.startup.retainassign": "true",
"site.fs.defaultFS": "hdfs://cluster:8020",
"site.fs.default.name": "hdfs://cluster:8020",
"site.hbase.master.info.port": "0",
"site.hbase.regionserver.info.port": "0",
"jvm.heapsize": "512M"
}
}
}
]
}

View File

@ -1,20 +1,12 @@
{
"name": "app-1",
"lifetime": "3600",
"launch_command": "sleep 3600",
"configuration": {
"properties": {
"g1": "a",
"g2": "b",
"internal.chaos.monkey.interval.seconds": "60",
"zookeeper.port": "2181",
"zookeeper.path": "/yarnapps_small_cluster",
"zookeeper.hosts": "zoo1,zoo2,zoo3",
"env.MALLOC_ARENA_MAX": "4",
"site.hbase.master.startup.retainassign": "true",
"site.fs.defaultFS": "hdfs://cluster:8020",
"site.fs.default.name": "hdfs://cluster:8020",
"site.hbase.master.info.port": "0",
"site.hbase.regionserver.info.port": "0"
"internal.chaos.monkey.interval.seconds": "60"
}
},
"resource": {
@ -32,6 +24,7 @@
"configuration": {
"properties": {
"g1": "overridden",
"g3": "will-be-overridden",
"jvm.heapsize": "512M"
}
}

View File

@ -0,0 +1,16 @@
{
"name": "default-app-1",
"lifetime": "3600",
"components" :
[
{
"name": "SLEEP",
"number_of_containers": 1,
"launch_command": "sleep 3600",
"resource": {
"cpus": 2,
"memory": "256"
}
}
]
}

View File

@ -0,0 +1,8 @@
{
"name": "external-0",
"lifetime": "3600",
"artifact": {
"type": "APPLICATION",
"id": "app-1"
}
}

View File

@ -0,0 +1,30 @@
{
"name": "external-1",
"lifetime": "3600",
"components": [
{
"name": "simple",
"artifact": {
"type": "APPLICATION",
"id": "app-1"
}
},
{
"name": "master",
"configuration": {
"properties": {
"g3": "is-overridden"
}
}
},
{
"name": "other",
"launch_command": "sleep 3600",
"number_of_containers": 2,
"resource": {
"cpus": 1,
"memory": "512"
}
}
]
}

View File

@ -0,0 +1,22 @@
{
"name": "external-2",
"lifetime": "3600",
"components": [
{
"name": "ext",
"artifact": {
"type": "APPLICATION",
"id": "external-1"
}
},
{
"name": "another",
"launch_command": "sleep 3600",
"number_of_containers": 1,
"resource": {
"cpus": 1,
"memory": "512"
}
}
]
}