Merge -c 1415029 from trunk to branch-2 to fix YARN-229. Remove old unused RM recovery code. Contributed by Bikas Saha.
git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-2@1415030 13f79535-47bb-0310-9956-ffa450edef68
This commit is contained in:
parent
b432cf076e
commit
7b092edd2f
|
@ -98,6 +98,8 @@ Release 2.0.3-alpha - Unreleased
|
||||||
MAPREDUCE-4778. Fair scheduler event log is only written if directory
|
MAPREDUCE-4778. Fair scheduler event log is only written if directory
|
||||||
exists on HDFS. (Sandy Ryza via tomwhite)
|
exists on HDFS. (Sandy Ryza via tomwhite)
|
||||||
|
|
||||||
|
YARN-229. Remove old unused RM recovery code. (Bikas Saha via acmurthy)
|
||||||
|
|
||||||
Release 2.0.2-alpha - 2012-09-07
|
Release 2.0.2-alpha - 2012-09-07
|
||||||
|
|
||||||
YARN-9. Rename YARN_HOME to HADOOP_YARN_HOME. (vinodkv via acmurthy)
|
YARN-9. Rename YARN_HOME to HADOOP_YARN_HOME. (vinodkv via acmurthy)
|
||||||
|
|
|
@ -28,7 +28,7 @@ import org.apache.hadoop.tools.GetGroupsTestBase;
|
||||||
import org.apache.hadoop.util.Tool;
|
import org.apache.hadoop.util.Tool;
|
||||||
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.service.Service.STATE;
|
import org.apache.hadoop.yarn.service.Service.STATE;
|
||||||
import org.junit.AfterClass;
|
import org.junit.AfterClass;
|
||||||
|
@ -46,7 +46,7 @@ public class TestGetGroups extends GetGroupsTestBase {
|
||||||
@BeforeClass
|
@BeforeClass
|
||||||
public static void setUpResourceManager() throws IOException, InterruptedException {
|
public static void setUpResourceManager() throws IOException, InterruptedException {
|
||||||
conf = new YarnConfiguration();
|
conf = new YarnConfiguration();
|
||||||
Store store = StoreFactory.getStore(conf);
|
RMStateStore store = StoreFactory.getStore(conf);
|
||||||
resourceManager = new ResourceManager(store) {
|
resourceManager = new ResourceManager(store) {
|
||||||
@Override
|
@Override
|
||||||
protected void doSecureLogin() throws IOException {
|
protected void doSecureLogin() throws IOException {
|
||||||
|
|
|
@ -228,16 +228,7 @@ public class YarnConfiguration extends Configuration {
|
||||||
|
|
||||||
/** The class to use as the persistent store.*/
|
/** The class to use as the persistent store.*/
|
||||||
public static final String RM_STORE = RM_PREFIX + "store.class";
|
public static final String RM_STORE = RM_PREFIX + "store.class";
|
||||||
|
|
||||||
/** The address of the zookeeper instance to use with ZK store.*/
|
|
||||||
public static final String RM_ZK_STORE_ADDRESS =
|
|
||||||
RM_PREFIX + "zookeeper-store.address";
|
|
||||||
|
|
||||||
/** The zookeeper session timeout for the zookeeper store.*/
|
|
||||||
public static final String RM_ZK_STORE_TIMEOUT_MS =
|
|
||||||
RM_PREFIX + "zookeeper-store.session.timeout-ms";
|
|
||||||
public static final int DEFAULT_RM_ZK_STORE_TIMEOUT_MS = 60000;
|
|
||||||
|
|
||||||
/** The maximum number of completed applications RM keeps. */
|
/** The maximum number of completed applications RM keeps. */
|
||||||
public static final String RM_MAX_COMPLETED_APPLICATIONS =
|
public static final String RM_MAX_COMPLETED_APPLICATIONS =
|
||||||
RM_PREFIX + "max-completed-applications";
|
RM_PREFIX + "max-completed-applications";
|
||||||
|
|
|
@ -209,17 +209,6 @@
|
||||||
<name>yarn.resourcemanager.store.class</name>
|
<name>yarn.resourcemanager.store.class</name>
|
||||||
</property>
|
</property>
|
||||||
|
|
||||||
<property>
|
|
||||||
<description>The address of the zookeeper instance to use with ZK store.</description>
|
|
||||||
<name>yarn.resourcemanager.zookeeper-store.address</name>
|
|
||||||
</property>
|
|
||||||
|
|
||||||
<property>
|
|
||||||
<description>The zookeeper session timeout for the zookeeper store.</description>
|
|
||||||
<name>yarn.resourcemanager.zookeeper-store.session.timeout-ms</name>
|
|
||||||
<value>60000</value>
|
|
||||||
</property>
|
|
||||||
|
|
||||||
<property>
|
<property>
|
||||||
<description>The maximum number of completed applications RM keeps. </description>
|
<description>The maximum number of completed applications RM keeps. </description>
|
||||||
<name>yarn.resourcemanager.max-completed-applications</name>
|
<name>yarn.resourcemanager.max-completed-applications</name>
|
||||||
|
|
|
@ -36,7 +36,6 @@ import org.apache.hadoop.yarn.event.EventHandler;
|
||||||
import org.apache.hadoop.yarn.ipc.RPCUtil;
|
import org.apache.hadoop.yarn.ipc.RPCUtil;
|
||||||
import org.apache.hadoop.yarn.security.client.ClientTokenIdentifier;
|
import org.apache.hadoop.yarn.security.client.ClientTokenIdentifier;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEventType;
|
||||||
|
@ -251,17 +250,12 @@ public class RMAppManager implements EventHandler<RMAppManagerEvent> {
|
||||||
YarnConfiguration.DEFAULT_APPLICATION_NAME);
|
YarnConfiguration.DEFAULT_APPLICATION_NAME);
|
||||||
}
|
}
|
||||||
|
|
||||||
// Store application for recovery
|
|
||||||
ApplicationStore appStore = rmContext.getApplicationsStore()
|
|
||||||
.createApplicationStore(submissionContext.getApplicationId(),
|
|
||||||
submissionContext);
|
|
||||||
|
|
||||||
// Create RMApp
|
// Create RMApp
|
||||||
application =
|
application =
|
||||||
new RMAppImpl(applicationId, rmContext, this.conf,
|
new RMAppImpl(applicationId, rmContext, this.conf,
|
||||||
submissionContext.getApplicationName(),
|
submissionContext.getApplicationName(),
|
||||||
submissionContext.getUser(), submissionContext.getQueue(),
|
submissionContext.getUser(), submissionContext.getQueue(),
|
||||||
submissionContext, clientTokenStr, appStore, this.scheduler,
|
submissionContext, clientTokenStr, this.scheduler,
|
||||||
this.masterService, submitTime);
|
this.masterService, submitTime);
|
||||||
|
|
||||||
// Sanity check - duplicate?
|
// Sanity check - duplicate?
|
||||||
|
|
|
@ -23,8 +23,6 @@ import java.util.concurrent.ConcurrentMap;
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
||||||
import org.apache.hadoop.yarn.api.records.NodeId;
|
import org.apache.hadoop.yarn.api.records.NodeId;
|
||||||
import org.apache.hadoop.yarn.event.Dispatcher;
|
import org.apache.hadoop.yarn.event.Dispatcher;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.NodeStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.ContainerAllocationExpirer;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.ContainerAllocationExpirer;
|
||||||
|
@ -41,10 +39,6 @@ public interface RMContext {
|
||||||
|
|
||||||
Dispatcher getDispatcher();
|
Dispatcher getDispatcher();
|
||||||
|
|
||||||
NodeStore getNodeStore();
|
|
||||||
|
|
||||||
ApplicationsStore getApplicationsStore();
|
|
||||||
|
|
||||||
ConcurrentMap<ApplicationId, RMApp> getRMApps();
|
ConcurrentMap<ApplicationId, RMApp> getRMApps();
|
||||||
|
|
||||||
ConcurrentMap<String, RMNode> getInactiveRMNodes();
|
ConcurrentMap<String, RMNode> getInactiveRMNodes();
|
||||||
|
|
|
@ -24,9 +24,6 @@ import java.util.concurrent.ConcurrentMap;
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
||||||
import org.apache.hadoop.yarn.api.records.NodeId;
|
import org.apache.hadoop.yarn.api.records.NodeId;
|
||||||
import org.apache.hadoop.yarn.event.Dispatcher;
|
import org.apache.hadoop.yarn.event.Dispatcher;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.NodeStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.ContainerAllocationExpirer;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.ContainerAllocationExpirer;
|
||||||
|
@ -39,7 +36,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.security.RMContainerTokenSe
|
||||||
public class RMContextImpl implements RMContext {
|
public class RMContextImpl implements RMContext {
|
||||||
|
|
||||||
private final Dispatcher rmDispatcher;
|
private final Dispatcher rmDispatcher;
|
||||||
private final Store store;
|
|
||||||
|
|
||||||
private final ConcurrentMap<ApplicationId, RMApp> applications
|
private final ConcurrentMap<ApplicationId, RMApp> applications
|
||||||
= new ConcurrentHashMap<ApplicationId, RMApp>();
|
= new ConcurrentHashMap<ApplicationId, RMApp>();
|
||||||
|
@ -58,7 +54,7 @@ public class RMContextImpl implements RMContext {
|
||||||
private final RMContainerTokenSecretManager containerTokenSecretManager;
|
private final RMContainerTokenSecretManager containerTokenSecretManager;
|
||||||
private final ClientToAMTokenSecretManagerInRM clientToAMTokenSecretManager;
|
private final ClientToAMTokenSecretManagerInRM clientToAMTokenSecretManager;
|
||||||
|
|
||||||
public RMContextImpl(Store store, Dispatcher rmDispatcher,
|
public RMContextImpl(Dispatcher rmDispatcher,
|
||||||
ContainerAllocationExpirer containerAllocationExpirer,
|
ContainerAllocationExpirer containerAllocationExpirer,
|
||||||
AMLivelinessMonitor amLivelinessMonitor,
|
AMLivelinessMonitor amLivelinessMonitor,
|
||||||
AMLivelinessMonitor amFinishingMonitor,
|
AMLivelinessMonitor amFinishingMonitor,
|
||||||
|
@ -66,7 +62,6 @@ public class RMContextImpl implements RMContext {
|
||||||
ApplicationTokenSecretManager appTokenSecretManager,
|
ApplicationTokenSecretManager appTokenSecretManager,
|
||||||
RMContainerTokenSecretManager containerTokenSecretManager,
|
RMContainerTokenSecretManager containerTokenSecretManager,
|
||||||
ClientToAMTokenSecretManagerInRM clientTokenSecretManager) {
|
ClientToAMTokenSecretManagerInRM clientTokenSecretManager) {
|
||||||
this.store = store;
|
|
||||||
this.rmDispatcher = rmDispatcher;
|
this.rmDispatcher = rmDispatcher;
|
||||||
this.containerAllocationExpirer = containerAllocationExpirer;
|
this.containerAllocationExpirer = containerAllocationExpirer;
|
||||||
this.amLivelinessMonitor = amLivelinessMonitor;
|
this.amLivelinessMonitor = amLivelinessMonitor;
|
||||||
|
@ -82,16 +77,6 @@ public class RMContextImpl implements RMContext {
|
||||||
return this.rmDispatcher;
|
return this.rmDispatcher;
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
|
||||||
public NodeStore getNodeStore() {
|
|
||||||
return store;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public ApplicationsStore getApplicationsStore() {
|
|
||||||
return store;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public ConcurrentMap<ApplicationId, RMApp> getRMApps() {
|
public ConcurrentMap<ApplicationId, RMApp> getRMApps() {
|
||||||
return this.applications;
|
return this.applications;
|
||||||
|
|
|
@ -46,8 +46,8 @@ import org.apache.hadoop.yarn.server.RMDelegationTokenSecretManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.ApplicationMasterLauncher;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.ApplicationMasterLauncher;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Recoverable;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Recoverable;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store.RMState;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore.RMState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
||||||
|
@ -119,12 +119,12 @@ public class ResourceManager extends CompositeService implements Recoverable {
|
||||||
protected RMDelegationTokenSecretManager rmDTSecretManager;
|
protected RMDelegationTokenSecretManager rmDTSecretManager;
|
||||||
private WebApp webApp;
|
private WebApp webApp;
|
||||||
protected RMContext rmContext;
|
protected RMContext rmContext;
|
||||||
private final Store store;
|
private final RMStateStore store;
|
||||||
protected ResourceTrackerService resourceTracker;
|
protected ResourceTrackerService resourceTracker;
|
||||||
|
|
||||||
private Configuration conf;
|
private Configuration conf;
|
||||||
|
|
||||||
public ResourceManager(Store store) {
|
public ResourceManager(RMStateStore store) {
|
||||||
super("ResourceManager");
|
super("ResourceManager");
|
||||||
this.store = store;
|
this.store = store;
|
||||||
}
|
}
|
||||||
|
@ -161,7 +161,7 @@ public class ResourceManager extends CompositeService implements Recoverable {
|
||||||
this.containerTokenSecretManager = createContainerTokenSecretManager(conf);
|
this.containerTokenSecretManager = createContainerTokenSecretManager(conf);
|
||||||
|
|
||||||
this.rmContext =
|
this.rmContext =
|
||||||
new RMContextImpl(this.store, this.rmDispatcher,
|
new RMContextImpl(this.rmDispatcher,
|
||||||
this.containerAllocationExpirer, amLivelinessMonitor,
|
this.containerAllocationExpirer, amLivelinessMonitor,
|
||||||
amFinishingMonitor, tokenRenewer, this.appTokenSecretManager,
|
amFinishingMonitor, tokenRenewer, this.appTokenSecretManager,
|
||||||
this.containerTokenSecretManager, this.clientToAMSecretManager);
|
this.containerTokenSecretManager, this.clientToAMSecretManager);
|
||||||
|
@ -643,8 +643,6 @@ public class ResourceManager extends CompositeService implements Recoverable {
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public void recover(RMState state) throws Exception {
|
public void recover(RMState state) throws Exception {
|
||||||
resourceTracker.recover(state);
|
|
||||||
scheduler.recover(state);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
public static void main(String argv[]) {
|
public static void main(String argv[]) {
|
||||||
|
@ -652,14 +650,13 @@ public class ResourceManager extends CompositeService implements Recoverable {
|
||||||
StringUtils.startupShutdownMessage(ResourceManager.class, argv, LOG);
|
StringUtils.startupShutdownMessage(ResourceManager.class, argv, LOG);
|
||||||
try {
|
try {
|
||||||
Configuration conf = new YarnConfiguration();
|
Configuration conf = new YarnConfiguration();
|
||||||
Store store = StoreFactory.getStore(conf);
|
RMStateStore store = StoreFactory.getStore(conf);
|
||||||
ResourceManager resourceManager = new ResourceManager(store);
|
ResourceManager resourceManager = new ResourceManager(store);
|
||||||
ShutdownHookManager.get().addShutdownHook(
|
ShutdownHookManager.get().addShutdownHook(
|
||||||
new CompositeServiceShutdownHook(resourceManager),
|
new CompositeServiceShutdownHook(resourceManager),
|
||||||
SHUTDOWN_HOOK_PRIORITY);
|
SHUTDOWN_HOOK_PRIORITY);
|
||||||
resourceManager.init(conf);
|
resourceManager.init(conf);
|
||||||
//resourceManager.recover(store.restore());
|
//resourceManager.recover(store.restore());
|
||||||
//store.doneWithRecovery();
|
|
||||||
resourceManager.start();
|
resourceManager.start();
|
||||||
} catch (Throwable t) {
|
} catch (Throwable t) {
|
||||||
LOG.fatal("Error starting ResourceManager", t);
|
LOG.fatal("Error starting ResourceManager", t);
|
||||||
|
|
|
@ -44,7 +44,7 @@ import org.apache.hadoop.yarn.server.api.records.MasterKey;
|
||||||
import org.apache.hadoop.yarn.server.api.records.NodeAction;
|
import org.apache.hadoop.yarn.server.api.records.NodeAction;
|
||||||
import org.apache.hadoop.yarn.server.api.records.NodeStatus;
|
import org.apache.hadoop.yarn.server.api.records.NodeStatus;
|
||||||
import org.apache.hadoop.yarn.server.api.records.RegistrationResponse;
|
import org.apache.hadoop.yarn.server.api.records.RegistrationResponse;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store.RMState;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore.RMState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEventType;
|
||||||
|
@ -297,28 +297,6 @@ public class ResourceTrackerService extends AbstractService implements
|
||||||
return nodeHeartBeatResponse;
|
return nodeHeartBeatResponse;
|
||||||
}
|
}
|
||||||
|
|
||||||
public void recover(RMState state) {
|
|
||||||
//
|
|
||||||
// List<RMNode> nodeManagers = state.getStoredNodeManagers();
|
|
||||||
// for (RMNode nm : nodeManagers) {
|
|
||||||
// createNewNode(nm.getNodeID(), nm.getNodeHostName(), nm
|
|
||||||
// .getCommandPort(), nm.getHttpPort(), nm.getNode(), nm
|
|
||||||
// .getTotalCapability());
|
|
||||||
// }
|
|
||||||
// for (Map.Entry<ApplicationId, ApplicationInfo> entry : state
|
|
||||||
// .getStoredApplications().entrySet()) {
|
|
||||||
// List<Container> containers = entry.getValue().getContainers();
|
|
||||||
// List<Container> containersToAdd = new ArrayList<Container>();
|
|
||||||
// for (Container c : containers) {
|
|
||||||
// RMNode containerNode = this.rmContext.getNodesCollection()
|
|
||||||
// .getNodeInfo(c.getNodeId());
|
|
||||||
// containersToAdd.add(c);
|
|
||||||
// containerNode.allocateContainer(entry.getKey(), containersToAdd);
|
|
||||||
// containersToAdd.clear();
|
|
||||||
// }
|
|
||||||
// }
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* resolving the network topology.
|
* resolving the network topology.
|
||||||
* @param hostName the hostname of this node.
|
* @param hostName the hostname of this node.
|
||||||
|
|
|
@ -1,39 +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.server.resourcemanager.recovery;
|
|
||||||
|
|
||||||
import java.io.IOException;
|
|
||||||
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationMaster;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
|
|
||||||
import org.apache.hadoop.yarn.api.records.Container;
|
|
||||||
|
|
||||||
public interface ApplicationsStore {
|
|
||||||
public ApplicationStore createApplicationStore(ApplicationId applicationId,
|
|
||||||
ApplicationSubmissionContext context) throws IOException;
|
|
||||||
public void removeApplication(ApplicationId application) throws IOException;
|
|
||||||
|
|
||||||
public interface ApplicationStore {
|
|
||||||
public void storeContainer(Container container) throws IOException;
|
|
||||||
public void removeContainer(Container container) throws IOException;
|
|
||||||
public void storeMasterContainer(Container container) throws IOException;
|
|
||||||
public void updateApplicationState(ApplicationMaster master) throws IOException;
|
|
||||||
public boolean isLoggable();
|
|
||||||
}
|
|
||||||
}
|
|
|
@ -15,18 +15,8 @@
|
||||||
* See the License for the specific language governing permissions and
|
* See the License for the specific language governing permissions and
|
||||||
* limitations under the License.
|
* limitations under the License.
|
||||||
*/
|
*/
|
||||||
|
|
||||||
package org.apache.hadoop.yarn.server.resourcemanager.recovery;
|
package org.apache.hadoop.yarn.server.resourcemanager.recovery;
|
||||||
|
|
||||||
import java.io.IOException;
|
public class FileRMStateStore implements RMStateStore {
|
||||||
|
|
||||||
import org.apache.hadoop.yarn.api.records.NodeId;
|
}
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
|
||||||
|
|
||||||
|
|
||||||
public interface NodeStore {
|
|
||||||
public void storeNode(RMNode node) throws IOException;
|
|
||||||
public void removeNode(RMNode node) throws IOException;
|
|
||||||
public NodeId getNextNodeId() throws IOException;
|
|
||||||
public boolean isLoggable();
|
|
||||||
}
|
|
|
@ -1,128 +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.server.resourcemanager.recovery;
|
|
||||||
import java.io.IOException;
|
|
||||||
import java.util.ArrayList;
|
|
||||||
import java.util.HashMap;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.Map;
|
|
||||||
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationMaster;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
|
|
||||||
import org.apache.hadoop.yarn.api.records.Container;
|
|
||||||
import org.apache.hadoop.yarn.api.records.NodeId;
|
|
||||||
import org.apache.hadoop.yarn.factories.RecordFactory;
|
|
||||||
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
|
||||||
|
|
||||||
public class MemStore implements Store {
|
|
||||||
RecordFactory recordFactory = RecordFactoryProvider.getRecordFactory(null);
|
|
||||||
private NodeId nodeId;
|
|
||||||
private boolean doneWithRecovery = false;
|
|
||||||
|
|
||||||
public MemStore() {
|
|
||||||
nodeId = recordFactory.newRecordInstance(NodeId.class);
|
|
||||||
nodeId.setHost("TODO");
|
|
||||||
nodeId.setPort(-1);
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void storeNode(RMNode node) throws IOException {}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void removeNode(RMNode node) throws IOException {}
|
|
||||||
|
|
||||||
private class ApplicationStoreImpl implements ApplicationStore {
|
|
||||||
@Override
|
|
||||||
public void storeContainer(Container container) throws IOException {}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void removeContainer(Container container) throws IOException {}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void storeMasterContainer(Container container) throws IOException {}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void updateApplicationState(
|
|
||||||
ApplicationMaster master) throws IOException {}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public boolean isLoggable() {
|
|
||||||
return doneWithRecovery;
|
|
||||||
}
|
|
||||||
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public ApplicationStore createApplicationStore(ApplicationId application,
|
|
||||||
ApplicationSubmissionContext context) throws IOException {
|
|
||||||
return new ApplicationStoreImpl();
|
|
||||||
}
|
|
||||||
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void removeApplication(ApplicationId application) throws IOException {}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public RMState restore() throws IOException {
|
|
||||||
MemRMState state = new MemRMState();
|
|
||||||
return state;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public synchronized NodeId getNextNodeId() throws IOException {
|
|
||||||
// TODO: FIXMEVinodkv
|
|
||||||
// int num = nodeId.getId();
|
|
||||||
// num++;
|
|
||||||
// nodeId.setId(num);
|
|
||||||
return nodeId;
|
|
||||||
}
|
|
||||||
|
|
||||||
private class MemRMState implements RMState {
|
|
||||||
|
|
||||||
public MemRMState() {
|
|
||||||
nodeId = recordFactory.newRecordInstance(NodeId.class);
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public List<RMNode> getStoredNodeManagers() {
|
|
||||||
return new ArrayList<RMNode>();
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public NodeId getLastLoggedNodeId() {
|
|
||||||
return nodeId;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public Map<ApplicationId, ApplicationInfo> getStoredApplications() {
|
|
||||||
return new HashMap<ApplicationId, Store.ApplicationInfo>();
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public boolean isLoggable() {
|
|
||||||
return doneWithRecovery;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void doneWithRecovery() {
|
|
||||||
doneWithRecovery = true;
|
|
||||||
}
|
|
||||||
}
|
|
|
@ -0,0 +1,24 @@
|
||||||
|
/**
|
||||||
|
* Licensed to the Apache Software Foundation (ASF) under one
|
||||||
|
* or more contributor license agreements. See the NOTICE file
|
||||||
|
* distributed with this work for additional information
|
||||||
|
* regarding copyright ownership. The ASF licenses this file
|
||||||
|
* to you under the Apache License, Version 2.0 (the
|
||||||
|
* "License"); you may not use this file except in compliance
|
||||||
|
* with the License. You may obtain a copy of the License at
|
||||||
|
*
|
||||||
|
* http://www.apache.org/licenses/LICENSE-2.0
|
||||||
|
*
|
||||||
|
* Unless required by applicable law or agreed to in writing, software
|
||||||
|
* distributed under the License is distributed on an "AS IS" BASIS,
|
||||||
|
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||||
|
* See the License for the specific language governing permissions and
|
||||||
|
* limitations under the License.
|
||||||
|
*/
|
||||||
|
package org.apache.hadoop.yarn.server.resourcemanager.recovery;
|
||||||
|
|
||||||
|
public interface RMStateStore {
|
||||||
|
public interface RMState {
|
||||||
|
|
||||||
|
}
|
||||||
|
}
|
|
@ -17,7 +17,7 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.hadoop.yarn.server.resourcemanager.recovery;
|
package org.apache.hadoop.yarn.server.resourcemanager.recovery;
|
||||||
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store.RMState;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore.RMState;
|
||||||
|
|
||||||
public interface Recoverable {
|
public interface Recoverable {
|
||||||
public void recover(RMState state) throws Exception;
|
public void recover(RMState state) throws Exception;
|
||||||
|
|
|
@ -1,46 +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.server.resourcemanager.recovery;
|
|
||||||
|
|
||||||
import java.io.IOException;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.Map;
|
|
||||||
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationMaster;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
|
|
||||||
import org.apache.hadoop.yarn.api.records.Container;
|
|
||||||
import org.apache.hadoop.yarn.api.records.NodeId;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
|
||||||
|
|
||||||
|
|
||||||
public interface Store extends NodeStore, ApplicationsStore {
|
|
||||||
public interface ApplicationInfo {
|
|
||||||
public ApplicationMaster getApplicationMaster();
|
|
||||||
public Container getMasterContainer();
|
|
||||||
public ApplicationSubmissionContext getApplicationSubmissionContext();
|
|
||||||
public List<Container> getContainers();
|
|
||||||
}
|
|
||||||
public interface RMState {
|
|
||||||
public List<RMNode> getStoredNodeManagers() ;
|
|
||||||
public Map<ApplicationId, ApplicationInfo> getStoredApplications();
|
|
||||||
public NodeId getLastLoggedNodeId();
|
|
||||||
}
|
|
||||||
public RMState restore() throws IOException;
|
|
||||||
public void doneWithRecovery();
|
|
||||||
}
|
|
|
@ -17,53 +17,17 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.hadoop.yarn.server.resourcemanager.recovery;
|
package org.apache.hadoop.yarn.server.resourcemanager.recovery;
|
||||||
|
|
||||||
import java.io.IOException;
|
|
||||||
|
|
||||||
import org.apache.hadoop.conf.Configuration;
|
import org.apache.hadoop.conf.Configuration;
|
||||||
import org.apache.hadoop.util.ReflectionUtils;
|
import org.apache.hadoop.util.ReflectionUtils;
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationMaster;
|
|
||||||
import org.apache.hadoop.yarn.api.records.Container;
|
|
||||||
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
|
|
||||||
public class StoreFactory {
|
public class StoreFactory {
|
||||||
|
|
||||||
public static Store getStore(Configuration conf) {
|
public static RMStateStore getStore(Configuration conf) {
|
||||||
Store store = ReflectionUtils.newInstance(
|
RMStateStore store = ReflectionUtils.newInstance(
|
||||||
conf.getClass(YarnConfiguration.RM_STORE,
|
conf.getClass(YarnConfiguration.RM_STORE,
|
||||||
MemStore.class, Store.class),
|
FileRMStateStore.class, RMStateStore.class),
|
||||||
conf);
|
conf);
|
||||||
return store;
|
return store;
|
||||||
}
|
}
|
||||||
|
|
||||||
public static ApplicationStore createVoidAppStore() {
|
|
||||||
return new VoidApplicationStore();
|
|
||||||
}
|
|
||||||
|
|
||||||
private static class VoidApplicationStore implements ApplicationStore {
|
|
||||||
|
|
||||||
public VoidApplicationStore() {}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void storeContainer(Container container) throws IOException {
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void removeContainer(Container container) throws IOException {
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void storeMasterContainer(Container container) throws IOException {
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void updateApplicationState(ApplicationMaster master)
|
|
||||||
throws IOException {
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public boolean isLoggable() {
|
|
||||||
return false;
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
}
|
|
@ -1,509 +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.server.resourcemanager.recovery;
|
|
||||||
|
|
||||||
import java.io.IOException;
|
|
||||||
import java.io.InterruptedIOException;
|
|
||||||
import java.util.ArrayList;
|
|
||||||
import java.util.HashMap;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.Map;
|
|
||||||
import java.util.regex.Matcher;
|
|
||||||
import java.util.regex.Pattern;
|
|
||||||
|
|
||||||
import org.apache.commons.logging.Log;
|
|
||||||
import org.apache.commons.logging.LogFactory;
|
|
||||||
import org.apache.hadoop.conf.Configuration;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationMaster;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
|
|
||||||
import org.apache.hadoop.yarn.api.records.Container;
|
|
||||||
import org.apache.hadoop.yarn.api.records.ContainerId;
|
|
||||||
import org.apache.hadoop.yarn.api.records.NodeId;
|
|
||||||
import org.apache.hadoop.yarn.api.records.NodeReport;
|
|
||||||
import org.apache.hadoop.yarn.api.records.impl.pb.ApplicationMasterPBImpl;
|
|
||||||
import org.apache.hadoop.yarn.api.records.impl.pb.ApplicationSubmissionContextPBImpl;
|
|
||||||
import org.apache.hadoop.yarn.api.records.impl.pb.ContainerPBImpl;
|
|
||||||
import org.apache.hadoop.yarn.api.records.impl.pb.NodeIdPBImpl;
|
|
||||||
import org.apache.hadoop.yarn.api.records.impl.pb.NodeReportPBImpl;
|
|
||||||
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
||||||
import org.apache.hadoop.yarn.factories.RecordFactory;
|
|
||||||
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
|
||||||
import org.apache.hadoop.yarn.proto.YarnProtos.ApplicationMasterProto;
|
|
||||||
import org.apache.hadoop.yarn.proto.YarnProtos.ApplicationSubmissionContextProto;
|
|
||||||
import org.apache.hadoop.yarn.proto.YarnProtos.ContainerProto;
|
|
||||||
import org.apache.hadoop.yarn.proto.YarnProtos.NodeIdProto;
|
|
||||||
import org.apache.hadoop.yarn.proto.YarnProtos.NodeReportProto;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
|
||||||
import org.apache.hadoop.yarn.util.ConverterUtils;
|
|
||||||
import org.apache.zookeeper.CreateMode;
|
|
||||||
import org.apache.zookeeper.KeeperException;
|
|
||||||
import org.apache.zookeeper.WatchedEvent;
|
|
||||||
import org.apache.zookeeper.Watcher;
|
|
||||||
import org.apache.zookeeper.ZooKeeper;
|
|
||||||
import org.apache.zookeeper.data.Stat;
|
|
||||||
|
|
||||||
public class ZKStore implements Store {
|
|
||||||
private final Configuration conf;
|
|
||||||
private final ZooKeeper zkClient;
|
|
||||||
private static final Log LOG = LogFactory.getLog(ZKStore.class);
|
|
||||||
private final RecordFactory recordFactory = RecordFactoryProvider.getRecordFactory(null);
|
|
||||||
private static final String NODES = "nodes/";
|
|
||||||
private static final String APPS = "apps/";
|
|
||||||
private static final String ZK_PATH_SEPARATOR = "/";
|
|
||||||
private static final String NODE_ID = "nodeid";
|
|
||||||
private static final String APP_MASTER = "master";
|
|
||||||
private static final String APP_MASTER_CONTAINER = "mastercontainer";
|
|
||||||
private final String ZK_ADDRESS;
|
|
||||||
private final int ZK_TIMEOUT;
|
|
||||||
private boolean doneWithRecovery = false;
|
|
||||||
|
|
||||||
/** TODO make this generic **/
|
|
||||||
private NodeIdPBImpl nodeId = new NodeIdPBImpl();
|
|
||||||
|
|
||||||
/**
|
|
||||||
* TODO fix this for later to handle all kinds of events
|
|
||||||
* of connection and session events.
|
|
||||||
*
|
|
||||||
*/
|
|
||||||
private static class ZKWatcher implements Watcher {
|
|
||||||
@Override
|
|
||||||
public void process(WatchedEvent arg0) {
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
public ZKStore(Configuration conf) throws IOException {
|
|
||||||
this.conf = conf;
|
|
||||||
this.ZK_ADDRESS = conf.get(YarnConfiguration.RM_ZK_STORE_ADDRESS);
|
|
||||||
this.ZK_TIMEOUT = conf.getInt(YarnConfiguration.RM_ZK_STORE_TIMEOUT_MS,
|
|
||||||
YarnConfiguration.DEFAULT_RM_ZK_STORE_TIMEOUT_MS);
|
|
||||||
zkClient = new ZooKeeper(this.ZK_ADDRESS,
|
|
||||||
this.ZK_TIMEOUT,
|
|
||||||
createZKWatcher()
|
|
||||||
);
|
|
||||||
// TODO: FIXMEVinodkv
|
|
||||||
// this.nodeId.setId(0);
|
|
||||||
}
|
|
||||||
|
|
||||||
protected Watcher createZKWatcher() {
|
|
||||||
return new ZKWatcher();
|
|
||||||
}
|
|
||||||
|
|
||||||
private NodeReportPBImpl createNodeManagerInfo(RMNode rmNode) {
|
|
||||||
NodeReport node =
|
|
||||||
recordFactory.newRecordInstance(NodeReport.class);
|
|
||||||
node.setNodeId(rmNode.getNodeID());
|
|
||||||
node.setRackName(rmNode.getRackName());
|
|
||||||
node.setCapability(rmNode.getTotalCapability());
|
|
||||||
// TODO: FIXME
|
|
||||||
// node.setUsed(nodeInfo.getUsedResource());
|
|
||||||
// TODO: acm: refactor2 FIXME
|
|
||||||
// node.setNumContainers(rmNode.getNumContainers());
|
|
||||||
return (NodeReportPBImpl)node;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public synchronized void storeNode(RMNode node) throws IOException {
|
|
||||||
/** create a storage node and store it in zk **/
|
|
||||||
if (!doneWithRecovery) return;
|
|
||||||
// TODO: FIXMEVinodkv
|
|
||||||
// NodeReportPBImpl nodeManagerInfo = createNodeManagerInfo(node);
|
|
||||||
// byte[] bytes = nodeManagerInfo.getProto().toByteArray();
|
|
||||||
// try {
|
|
||||||
// zkClient.create(NODES + Integer.toString(node.getNodeID().getId()), bytes, null,
|
|
||||||
// CreateMode.PERSISTENT);
|
|
||||||
// } catch(InterruptedException ie) {
|
|
||||||
// LOG.info("Interrupted", ie);
|
|
||||||
// throw new InterruptedIOException("Interrupted");
|
|
||||||
// } catch(KeeperException ke) {
|
|
||||||
// LOG.info("Keeper exception", ke);
|
|
||||||
// throw convertToIOException(ke);
|
|
||||||
// }
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public synchronized void removeNode(RMNode node) throws IOException {
|
|
||||||
if (!doneWithRecovery) return;
|
|
||||||
|
|
||||||
// TODO: FIXME VINODKV
|
|
||||||
// /** remove a storage node **/
|
|
||||||
// try {
|
|
||||||
// zkClient.delete(NODES + Integer.toString(node.getNodeID().getId()), -1);
|
|
||||||
// } catch(InterruptedException ie) {
|
|
||||||
// LOG.info("Interrupted", ie);
|
|
||||||
// throw new InterruptedIOException("Interrupted");
|
|
||||||
// } catch(KeeperException ke) {
|
|
||||||
// LOG.info("Keeper exception", ke);
|
|
||||||
// throw convertToIOException(ke);
|
|
||||||
// }
|
|
||||||
|
|
||||||
}
|
|
||||||
|
|
||||||
private static IOException convertToIOException(KeeperException ke) {
|
|
||||||
IOException io = new IOException();
|
|
||||||
io.setStackTrace(ke.getStackTrace());
|
|
||||||
return io;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public synchronized NodeId getNextNodeId() throws IOException {
|
|
||||||
// TODO: FIXME VINODKV
|
|
||||||
// int num = nodeId.getId();
|
|
||||||
// num++;
|
|
||||||
// nodeId.setId(num);
|
|
||||||
// try {
|
|
||||||
// zkClient.setData(NODES + NODE_ID, nodeId.getProto().toByteArray() , -1);
|
|
||||||
// } catch(InterruptedException ie) {
|
|
||||||
// LOG.info("Interrupted", ie);
|
|
||||||
// throw new InterruptedIOException(ie.getMessage());
|
|
||||||
// } catch(KeeperException ke) {
|
|
||||||
// throw convertToIOException(ke);
|
|
||||||
// }
|
|
||||||
return nodeId;
|
|
||||||
}
|
|
||||||
|
|
||||||
private String containerPathFromContainerId(ContainerId containerId) {
|
|
||||||
String appString = ConverterUtils.toString(
|
|
||||||
containerId.getApplicationAttemptId().getApplicationId());
|
|
||||||
return appString + "/" + containerId.getId();
|
|
||||||
}
|
|
||||||
|
|
||||||
private class ZKApplicationStore implements ApplicationStore {
|
|
||||||
private final ApplicationId applicationId;
|
|
||||||
|
|
||||||
public ZKApplicationStore(ApplicationId applicationId) {
|
|
||||||
this.applicationId = applicationId;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void storeMasterContainer(Container container) throws IOException {
|
|
||||||
if (!doneWithRecovery) return;
|
|
||||||
|
|
||||||
ContainerPBImpl containerPBImpl = (ContainerPBImpl) container;
|
|
||||||
try {
|
|
||||||
zkClient.setData(APPS +
|
|
||||||
ConverterUtils.toString(
|
|
||||||
container.getId().getApplicationAttemptId().getApplicationId())
|
|
||||||
+
|
|
||||||
ZK_PATH_SEPARATOR + APP_MASTER_CONTAINER
|
|
||||||
, containerPBImpl.getProto().toByteArray(), -1);
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted", ie);
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
LOG.info("Keeper exception", ke);
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
@Override
|
|
||||||
public synchronized void storeContainer(Container container) throws IOException {
|
|
||||||
if (!doneWithRecovery) return;
|
|
||||||
|
|
||||||
ContainerPBImpl containerPBImpl = (ContainerPBImpl) container;
|
|
||||||
try {
|
|
||||||
zkClient.create(APPS + containerPathFromContainerId(container.getId())
|
|
||||||
, containerPBImpl.getProto().toByteArray(), null, CreateMode.PERSISTENT);
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted", ie);
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
LOG.info("Keeper exception", ke);
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public synchronized void removeContainer(Container container) throws IOException {
|
|
||||||
if (!doneWithRecovery) return;
|
|
||||||
try {
|
|
||||||
zkClient.delete(APPS + containerPathFromContainerId(container.getId()),
|
|
||||||
-1);
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
LOG.info("Keeper exception", ke);
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void updateApplicationState(
|
|
||||||
ApplicationMaster master) throws IOException {
|
|
||||||
if (!doneWithRecovery) return;
|
|
||||||
|
|
||||||
String appString = APPS + ConverterUtils.toString(applicationId);
|
|
||||||
ApplicationMasterPBImpl masterPBImpl = (ApplicationMasterPBImpl) master;
|
|
||||||
try {
|
|
||||||
zkClient.setData(appString, masterPBImpl.getProto().toByteArray(), -1);
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted", ie);
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
LOG.info("Keeper exception", ke);
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public boolean isLoggable() {
|
|
||||||
return doneWithRecovery;
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public synchronized ApplicationStore createApplicationStore(ApplicationId application,
|
|
||||||
ApplicationSubmissionContext context) throws IOException {
|
|
||||||
if (!doneWithRecovery) return new ZKApplicationStore(application);
|
|
||||||
|
|
||||||
ApplicationSubmissionContextPBImpl contextPBImpl = (ApplicationSubmissionContextPBImpl) context;
|
|
||||||
String appString = APPS + ConverterUtils.toString(application);
|
|
||||||
|
|
||||||
ApplicationMasterPBImpl masterPBImpl = new ApplicationMasterPBImpl();
|
|
||||||
ContainerPBImpl container = new ContainerPBImpl();
|
|
||||||
try {
|
|
||||||
zkClient.create(appString, contextPBImpl.getProto()
|
|
||||||
.toByteArray(), null, CreateMode.PERSISTENT);
|
|
||||||
zkClient.create(appString + ZK_PATH_SEPARATOR + APP_MASTER,
|
|
||||||
masterPBImpl.getProto().toByteArray(), null, CreateMode.PERSISTENT);
|
|
||||||
zkClient.create(appString + ZK_PATH_SEPARATOR + APP_MASTER_CONTAINER,
|
|
||||||
container.getProto().toByteArray(), null, CreateMode.PERSISTENT);
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted", ie);
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
LOG.info("Keeper exception", ke);
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
return new ZKApplicationStore(application);
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public synchronized void removeApplication(ApplicationId application) throws IOException {
|
|
||||||
if (!doneWithRecovery) return;
|
|
||||||
|
|
||||||
try {
|
|
||||||
zkClient.delete(APPS + ConverterUtils.toString(application), -1);
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted", ie);
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
LOG.info("Keeper Exception", ke);
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public boolean isLoggable() {
|
|
||||||
return doneWithRecovery;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void doneWithRecovery() {
|
|
||||||
this.doneWithRecovery = true;
|
|
||||||
}
|
|
||||||
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public synchronized RMState restore() throws IOException {
|
|
||||||
ZKRMState rmState = new ZKRMState();
|
|
||||||
rmState.load();
|
|
||||||
return rmState;
|
|
||||||
}
|
|
||||||
|
|
||||||
private static class ApplicationInfoImpl implements ApplicationInfo {
|
|
||||||
private ApplicationMaster master;
|
|
||||||
private Container masterContainer;
|
|
||||||
|
|
||||||
private final ApplicationSubmissionContext context;
|
|
||||||
private final List<Container> containers = new ArrayList<Container>();
|
|
||||||
|
|
||||||
public ApplicationInfoImpl(ApplicationSubmissionContext context) {
|
|
||||||
this.context = context;
|
|
||||||
}
|
|
||||||
|
|
||||||
public void setApplicationMaster(ApplicationMaster master) {
|
|
||||||
this.master = master;
|
|
||||||
}
|
|
||||||
|
|
||||||
public void setMasterContainer(Container container) {
|
|
||||||
this.masterContainer = container;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public ApplicationMaster getApplicationMaster() {
|
|
||||||
return this.master;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public ApplicationSubmissionContext getApplicationSubmissionContext() {
|
|
||||||
return this.context;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public Container getMasterContainer() {
|
|
||||||
return this.masterContainer;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public List<Container> getContainers() {
|
|
||||||
return this.containers;
|
|
||||||
}
|
|
||||||
|
|
||||||
public void addContainer(Container container) {
|
|
||||||
containers.add(container);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
private class ZKRMState implements RMState {
|
|
||||||
private List<RMNode> nodeManagers = new ArrayList<RMNode>();
|
|
||||||
private Map<ApplicationId, ApplicationInfo> applications = new
|
|
||||||
HashMap<ApplicationId, ApplicationInfo>();
|
|
||||||
|
|
||||||
public ZKRMState() {
|
|
||||||
LOG.info("Restoring RM state from ZK");
|
|
||||||
}
|
|
||||||
|
|
||||||
private synchronized List<NodeReport> listStoredNodes() throws IOException {
|
|
||||||
/** get the list of nodes stored in zk **/
|
|
||||||
//TODO PB
|
|
||||||
List<NodeReport> nodes = new ArrayList<NodeReport>();
|
|
||||||
Stat stat = new Stat();
|
|
||||||
try {
|
|
||||||
List<String> children = zkClient.getChildren(NODES, false);
|
|
||||||
for (String child: children) {
|
|
||||||
byte[] data = zkClient.getData(NODES + child, false, stat);
|
|
||||||
NodeReportPBImpl nmImpl = new NodeReportPBImpl(
|
|
||||||
NodeReportProto.parseFrom(data));
|
|
||||||
nodes.add(nmImpl);
|
|
||||||
}
|
|
||||||
} catch (InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted" , ie);
|
|
||||||
throw new InterruptedIOException("Interrupted");
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
LOG.error("Failed to list nodes", ke);
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
return nodes;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public List<RMNode> getStoredNodeManagers() {
|
|
||||||
return nodeManagers;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public NodeId getLastLoggedNodeId() {
|
|
||||||
return nodeId;
|
|
||||||
}
|
|
||||||
|
|
||||||
private void readLastNodeId() throws IOException {
|
|
||||||
Stat stat = new Stat();
|
|
||||||
try {
|
|
||||||
byte[] data = zkClient.getData(NODES + NODE_ID, false, stat);
|
|
||||||
nodeId = new NodeIdPBImpl(NodeIdProto.parseFrom(data));
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted", ie);
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
LOG.info("Keeper Exception", ke);
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
private ApplicationInfo getAppInfo(String app) throws IOException {
|
|
||||||
ApplicationInfoImpl info = null;
|
|
||||||
Stat stat = new Stat();
|
|
||||||
try {
|
|
||||||
ApplicationSubmissionContext context = null;
|
|
||||||
byte[] data = zkClient.getData(APPS + app, false, stat);
|
|
||||||
context = new ApplicationSubmissionContextPBImpl(
|
|
||||||
ApplicationSubmissionContextProto.parseFrom(data));
|
|
||||||
info = new ApplicationInfoImpl(context);
|
|
||||||
List<String> children = zkClient.getChildren(APPS + app, false, stat);
|
|
||||||
ApplicationMaster master = null;
|
|
||||||
for (String child: children) {
|
|
||||||
byte[] childdata = zkClient.getData(APPS + app + ZK_PATH_SEPARATOR + child, false, stat);
|
|
||||||
if (APP_MASTER.equals(child)) {
|
|
||||||
master = new ApplicationMasterPBImpl(ApplicationMasterProto.parseFrom(childdata));
|
|
||||||
info.setApplicationMaster(master);
|
|
||||||
} else if (APP_MASTER_CONTAINER.equals(child)) {
|
|
||||||
Container masterContainer = new ContainerPBImpl(ContainerProto.parseFrom(data));
|
|
||||||
info.setMasterContainer(masterContainer);
|
|
||||||
} else {
|
|
||||||
Container container = new ContainerPBImpl(ContainerProto.parseFrom(data));
|
|
||||||
info.addContainer(container);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted", ie);
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
return info;
|
|
||||||
}
|
|
||||||
|
|
||||||
private void load() throws IOException {
|
|
||||||
List<NodeReport> nodeInfos = listStoredNodes();
|
|
||||||
final Pattern trackerPattern = Pattern.compile(".*:.*");
|
|
||||||
final Matcher m = trackerPattern.matcher("");
|
|
||||||
for (NodeReport node: nodeInfos) {
|
|
||||||
m.reset(node.getNodeId().getHost());
|
|
||||||
if (!m.find()) {
|
|
||||||
LOG.info("Skipping node, bad node-address "
|
|
||||||
+ node.getNodeId().getHost());
|
|
||||||
continue;
|
|
||||||
}
|
|
||||||
String hostName = m.group(0);
|
|
||||||
int cmPort = Integer.valueOf(m.group(1));
|
|
||||||
m.reset(node.getHttpAddress());
|
|
||||||
if (!m.find()) {
|
|
||||||
LOG.info("Skipping node, bad http-address " + node.getHttpAddress());
|
|
||||||
continue;
|
|
||||||
}
|
|
||||||
int httpPort = Integer.valueOf(m.group(1));
|
|
||||||
// TODO: FindBugs warns passing null below. Commenting this for later.
|
|
||||||
// RMNode nm = new RMNodeImpl(node.getNodeId(), null,
|
|
||||||
// hostName, cmPort, httpPort,
|
|
||||||
// ResourceTrackerService.resolve(node.getNodeId().getHost()),
|
|
||||||
// node.getCapability());
|
|
||||||
// nodeManagers.add(nm);
|
|
||||||
}
|
|
||||||
readLastNodeId();
|
|
||||||
/* make sure we get all the applications */
|
|
||||||
List<String> apps = null;
|
|
||||||
try {
|
|
||||||
apps = zkClient.getChildren(APPS, false);
|
|
||||||
} catch(InterruptedException ie) {
|
|
||||||
LOG.info("Interrupted", ie);
|
|
||||||
throw new InterruptedIOException(ie.getMessage());
|
|
||||||
} catch(KeeperException ke) {
|
|
||||||
throw convertToIOException(ke);
|
|
||||||
}
|
|
||||||
for (String app: apps) {
|
|
||||||
ApplicationInfo info = getAppInfo(app);
|
|
||||||
applications.put(info.getApplicationMaster().getApplicationId(), info);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public Map<ApplicationId, ApplicationInfo> getStoredApplications() {
|
|
||||||
return applications;
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
|
@ -0,0 +1,23 @@
|
||||||
|
/*
|
||||||
|
* 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.
|
||||||
|
*/
|
||||||
|
@InterfaceAudience.Private
|
||||||
|
@InterfaceStability.Unstable
|
||||||
|
package org.apache.hadoop.yarn.server.resourcemanager.recovery;
|
||||||
|
|
||||||
|
import org.apache.hadoop.classification.InterfaceAudience;
|
||||||
|
import org.apache.hadoop.classification.InterfaceStability;
|
|
@ -29,7 +29,6 @@ import org.apache.hadoop.yarn.api.records.ApplicationId;
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationReport;
|
import org.apache.hadoop.yarn.api.records.ApplicationReport;
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
|
import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
|
||||||
import org.apache.hadoop.yarn.event.EventHandler;
|
import org.apache.hadoop.yarn.event.EventHandler;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
||||||
|
|
||||||
|
@ -131,13 +130,6 @@ public interface RMApp extends EventHandler<RMAppEvent> {
|
||||||
*/
|
*/
|
||||||
int pullRMNodeUpdates(Collection<RMNode> updatedNodes);
|
int pullRMNodeUpdates(Collection<RMNode> updatedNodes);
|
||||||
|
|
||||||
/**
|
|
||||||
* Application level metadata is stored in {@link ApplicationStore} which
|
|
||||||
* can persist the information.
|
|
||||||
* @return the {@link ApplicationStore} for this {@link RMApp}.
|
|
||||||
*/
|
|
||||||
ApplicationStore getApplicationStore();
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* The finish time of the {@link RMApp}
|
* The finish time of the {@link RMApp}
|
||||||
* @return the finish time of the application.,
|
* @return the finish time of the application.,
|
||||||
|
|
|
@ -49,7 +49,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.ApplicationMasterService;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAppManagerEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAppManagerEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAppManagerEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAppManagerEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppNodeUpdateEvent.RMAppNodeUpdateType;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppNodeUpdateEvent.RMAppNodeUpdateType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
||||||
|
@ -81,7 +80,6 @@ public class RMAppImpl implements RMApp {
|
||||||
private final String name;
|
private final String name;
|
||||||
private final ApplicationSubmissionContext submissionContext;
|
private final ApplicationSubmissionContext submissionContext;
|
||||||
private final String clientTokenStr;
|
private final String clientTokenStr;
|
||||||
private final ApplicationStore appStore;
|
|
||||||
private final Dispatcher dispatcher;
|
private final Dispatcher dispatcher;
|
||||||
private final YarnScheduler scheduler;
|
private final YarnScheduler scheduler;
|
||||||
private final ApplicationMasterService masterService;
|
private final ApplicationMasterService masterService;
|
||||||
|
@ -213,7 +211,6 @@ public class RMAppImpl implements RMApp {
|
||||||
public RMAppImpl(ApplicationId applicationId, RMContext rmContext,
|
public RMAppImpl(ApplicationId applicationId, RMContext rmContext,
|
||||||
Configuration config, String name, String user, String queue,
|
Configuration config, String name, String user, String queue,
|
||||||
ApplicationSubmissionContext submissionContext, String clientTokenStr,
|
ApplicationSubmissionContext submissionContext, String clientTokenStr,
|
||||||
ApplicationStore appStore,
|
|
||||||
YarnScheduler scheduler, ApplicationMasterService masterService,
|
YarnScheduler scheduler, ApplicationMasterService masterService,
|
||||||
long submitTime) {
|
long submitTime) {
|
||||||
|
|
||||||
|
@ -227,7 +224,6 @@ public class RMAppImpl implements RMApp {
|
||||||
this.queue = queue;
|
this.queue = queue;
|
||||||
this.submissionContext = submissionContext;
|
this.submissionContext = submissionContext;
|
||||||
this.clientTokenStr = clientTokenStr;
|
this.clientTokenStr = clientTokenStr;
|
||||||
this.appStore = appStore;
|
|
||||||
this.scheduler = scheduler;
|
this.scheduler = scheduler;
|
||||||
this.masterService = masterService;
|
this.masterService = masterService;
|
||||||
this.submitTime = submitTime;
|
this.submitTime = submitTime;
|
||||||
|
@ -340,11 +336,6 @@ public class RMAppImpl implements RMApp {
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
|
||||||
public ApplicationStore getApplicationStore() {
|
|
||||||
return this.appStore;
|
|
||||||
}
|
|
||||||
|
|
||||||
private YarnApplicationState createApplicationState(RMAppState rmAppState) {
|
private YarnApplicationState createApplicationState(RMAppState rmAppState) {
|
||||||
switch(rmAppState) {
|
switch(rmAppState) {
|
||||||
case NEW:
|
case NEW:
|
||||||
|
|
|
@ -36,7 +36,6 @@ import org.apache.hadoop.yarn.api.records.Container;
|
||||||
import org.apache.hadoop.yarn.api.records.Priority;
|
import org.apache.hadoop.yarn.api.records.Priority;
|
||||||
import org.apache.hadoop.yarn.api.records.Resource;
|
import org.apache.hadoop.yarn.api.records.Resource;
|
||||||
import org.apache.hadoop.yarn.api.records.ResourceRequest;
|
import org.apache.hadoop.yarn.api.records.ResourceRequest;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
||||||
|
@ -70,14 +69,12 @@ public class AppSchedulingInfo {
|
||||||
boolean pending = true; // for app metrics
|
boolean pending = true; // for app metrics
|
||||||
|
|
||||||
public AppSchedulingInfo(ApplicationAttemptId appAttemptId,
|
public AppSchedulingInfo(ApplicationAttemptId appAttemptId,
|
||||||
String user, Queue queue, ActiveUsersManager activeUsersManager,
|
String user, Queue queue, ActiveUsersManager activeUsersManager) {
|
||||||
ApplicationStore store) {
|
|
||||||
this.applicationAttemptId = appAttemptId;
|
this.applicationAttemptId = appAttemptId;
|
||||||
this.applicationId = appAttemptId.getApplicationId();
|
this.applicationId = appAttemptId.getApplicationId();
|
||||||
this.queue = queue;
|
this.queue = queue;
|
||||||
this.queueName = queue.getQueueName();
|
this.queueName = queue.getQueueName();
|
||||||
this.user = user;
|
this.user = user;
|
||||||
//this.store = store;
|
|
||||||
this.activeUsersManager = activeUsersManager;
|
this.activeUsersManager = activeUsersManager;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
|
@ -49,7 +49,7 @@ import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store.RMState;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore.RMState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType;
|
||||||
|
@ -365,7 +365,7 @@ implements ResourceScheduler, CapacitySchedulerContext, Configurable {
|
||||||
// TODO: Fix store
|
// TODO: Fix store
|
||||||
FiCaSchedulerApp SchedulerApp =
|
FiCaSchedulerApp SchedulerApp =
|
||||||
new FiCaSchedulerApp(applicationAttemptId, user, queue,
|
new FiCaSchedulerApp(applicationAttemptId, user, queue,
|
||||||
queue.getActiveUsersManager(), rmContext, null);
|
queue.getActiveUsersManager(), rmContext);
|
||||||
|
|
||||||
// Submit to the queue
|
// Submit to the queue
|
||||||
try {
|
try {
|
||||||
|
@ -767,18 +767,7 @@ implements ResourceScheduler, CapacitySchedulerContext, Configurable {
|
||||||
@Override
|
@Override
|
||||||
@Lock(Lock.NoLock.class)
|
@Lock(Lock.NoLock.class)
|
||||||
public void recover(RMState state) throws Exception {
|
public void recover(RMState state) throws Exception {
|
||||||
// TODO: VINDOKVFIXME recovery
|
// NOT IMPLEMENTED
|
||||||
// applications.clear();
|
|
||||||
// for (Map.Entry<ApplicationId, ApplicationInfo> entry : state.getStoredApplications().entrySet()) {
|
|
||||||
// ApplicationId appId = entry.getKey();
|
|
||||||
// ApplicationInfo appInfo = entry.getValue();
|
|
||||||
// SchedulerApp app = applications.get(appId);
|
|
||||||
// app.allocate(appInfo.getContainers());
|
|
||||||
// for (Container c: entry.getValue().getContainers()) {
|
|
||||||
// Queue queue = queues.get(appInfo.getApplicationSubmissionContext().getQueue());
|
|
||||||
// queue.recoverContainer(clusterResource, applications.get(appId), c);
|
|
||||||
// }
|
|
||||||
// }
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
|
|
@ -43,7 +43,6 @@ import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainer;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainer;
|
||||||
|
@ -111,11 +110,11 @@ public class FiCaSchedulerApp extends SchedulerApplication {
|
||||||
private final RMContext rmContext;
|
private final RMContext rmContext;
|
||||||
public FiCaSchedulerApp(ApplicationAttemptId applicationAttemptId,
|
public FiCaSchedulerApp(ApplicationAttemptId applicationAttemptId,
|
||||||
String user, Queue queue, ActiveUsersManager activeUsersManager,
|
String user, Queue queue, ActiveUsersManager activeUsersManager,
|
||||||
RMContext rmContext, ApplicationStore store) {
|
RMContext rmContext) {
|
||||||
this.rmContext = rmContext;
|
this.rmContext = rmContext;
|
||||||
this.appSchedulingInfo =
|
this.appSchedulingInfo =
|
||||||
new AppSchedulingInfo(applicationAttemptId, user, queue,
|
new AppSchedulingInfo(applicationAttemptId, user, queue,
|
||||||
activeUsersManager, store);
|
activeUsersManager);
|
||||||
this.queue = queue;
|
this.queue = queue;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
|
@ -42,7 +42,6 @@ import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainer;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.RMContainer;
|
||||||
|
@ -103,11 +102,11 @@ public class FSSchedulerApp extends SchedulerApplication {
|
||||||
private final RMContext rmContext;
|
private final RMContext rmContext;
|
||||||
public FSSchedulerApp(ApplicationAttemptId applicationAttemptId,
|
public FSSchedulerApp(ApplicationAttemptId applicationAttemptId,
|
||||||
String user, Queue queue, ActiveUsersManager activeUsersManager,
|
String user, Queue queue, ActiveUsersManager activeUsersManager,
|
||||||
RMContext rmContext, ApplicationStore store) {
|
RMContext rmContext) {
|
||||||
this.rmContext = rmContext;
|
this.rmContext = rmContext;
|
||||||
this.appSchedulingInfo =
|
this.appSchedulingInfo =
|
||||||
new AppSchedulingInfo(applicationAttemptId, user, queue,
|
new AppSchedulingInfo(applicationAttemptId, user, queue,
|
||||||
activeUsersManager, store);
|
activeUsersManager);
|
||||||
this.queue = queue;
|
this.queue = queue;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
|
@ -50,7 +50,7 @@ import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store.RMState;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore.RMState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType;
|
||||||
|
@ -486,7 +486,7 @@ public class FairScheduler implements ResourceScheduler {
|
||||||
FSSchedulerApp schedulerApp =
|
FSSchedulerApp schedulerApp =
|
||||||
new FSSchedulerApp(applicationAttemptId, user,
|
new FSSchedulerApp(applicationAttemptId, user,
|
||||||
queue.getQueueSchedulable(), new ActiveUsersManager(getRootQueueMetrics()),
|
queue.getQueueSchedulable(), new ActiveUsersManager(getRootQueueMetrics()),
|
||||||
rmContext, null);
|
rmContext);
|
||||||
|
|
||||||
// Inforce ACLs
|
// Inforce ACLs
|
||||||
UserGroupInformation userUgi;
|
UserGroupInformation userUgi;
|
||||||
|
|
|
@ -56,7 +56,7 @@ import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAuditLogger.AuditConstants;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store.RMState;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore.RMState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType;
|
||||||
|
@ -292,7 +292,7 @@ public class FifoScheduler implements ResourceScheduler, Configurable {
|
||||||
// TODO: Fix store
|
// TODO: Fix store
|
||||||
FiCaSchedulerApp schedulerApp =
|
FiCaSchedulerApp schedulerApp =
|
||||||
new FiCaSchedulerApp(appAttemptId, user, DEFAULT_QUEUE, activeUsersManager,
|
new FiCaSchedulerApp(appAttemptId, user, DEFAULT_QUEUE, activeUsersManager,
|
||||||
this.rmContext, null);
|
this.rmContext);
|
||||||
applications.put(appAttemptId, schedulerApp);
|
applications.put(appAttemptId, schedulerApp);
|
||||||
metrics.submitApp(user, appAttemptId.getAttemptId());
|
metrics.submitApp(user, appAttemptId.getAttemptId());
|
||||||
LOG.info("Application Submission: " + appAttemptId.getApplicationId() +
|
LOG.info("Application Submission: " + appAttemptId.getApplicationId() +
|
||||||
|
@ -763,13 +763,7 @@ public class FifoScheduler implements ResourceScheduler, Configurable {
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public void recover(RMState state) {
|
public void recover(RMState state) {
|
||||||
// TODO fix recovery
|
// NOT IMPLEMENTED
|
||||||
// for (Map.Entry<ApplicationId, ApplicationInfo> entry: state.getStoredApplications().entrySet()) {
|
|
||||||
// ApplicationId appId = entry.getKey();
|
|
||||||
// ApplicationInfo appInfo = entry.getValue();
|
|
||||||
// SchedulerApp app = applications.get(appId);
|
|
||||||
// app.allocate(appInfo.getContainers());
|
|
||||||
// }
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
|
|
@ -37,7 +37,6 @@ import org.apache.hadoop.yarn.event.Dispatcher;
|
||||||
import org.apache.hadoop.yarn.event.EventHandler;
|
import org.apache.hadoop.yarn.event.EventHandler;
|
||||||
import org.apache.hadoop.yarn.factories.RecordFactory;
|
import org.apache.hadoop.yarn.factories.RecordFactory;
|
||||||
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.MockRMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.MockRMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
||||||
|
@ -93,7 +92,7 @@ public class TestAppManager{
|
||||||
rmDispatcher);
|
rmDispatcher);
|
||||||
AMLivelinessMonitor amFinishingMonitor = new AMLivelinessMonitor(
|
AMLivelinessMonitor amFinishingMonitor = new AMLivelinessMonitor(
|
||||||
rmDispatcher);
|
rmDispatcher);
|
||||||
return new RMContextImpl(new MemStore(), rmDispatcher,
|
return new RMContextImpl(rmDispatcher,
|
||||||
containerAllocationExpirer, amLivelinessMonitor, amFinishingMonitor,
|
containerAllocationExpirer, amLivelinessMonitor, amFinishingMonitor,
|
||||||
null, null, null, null) {
|
null, null, null, null) {
|
||||||
@Override
|
@Override
|
||||||
|
@ -366,7 +365,6 @@ public class TestAppManager{
|
||||||
YarnConfiguration.DEFAULT_QUEUE_NAME,
|
YarnConfiguration.DEFAULT_QUEUE_NAME,
|
||||||
app.getQueue());
|
app.getQueue());
|
||||||
Assert.assertEquals("app state doesn't match", RMAppState.NEW, app.getState());
|
Assert.assertEquals("app state doesn't match", RMAppState.NEW, app.getState());
|
||||||
Assert.assertNotNull("app store is null", app.getApplicationStore());
|
|
||||||
|
|
||||||
// wait for event to be processed
|
// wait for event to be processed
|
||||||
int timeoutSecs = 0;
|
int timeoutSecs = 0;
|
||||||
|
@ -413,7 +411,6 @@ public class TestAppManager{
|
||||||
Assert.assertEquals("app name doesn't match", "testApp1", app.getName());
|
Assert.assertEquals("app name doesn't match", "testApp1", app.getName());
|
||||||
Assert.assertEquals("app queue doesn't match", "testQueue", app.getQueue());
|
Assert.assertEquals("app queue doesn't match", "testQueue", app.getQueue());
|
||||||
Assert.assertEquals("app state doesn't match", RMAppState.NEW, app.getState());
|
Assert.assertEquals("app state doesn't match", RMAppState.NEW, app.getState());
|
||||||
Assert.assertNotNull("app store is null", app.getApplicationStore());
|
|
||||||
|
|
||||||
// wait for event to be processed
|
// wait for event to be processed
|
||||||
int timeoutSecs = 0;
|
int timeoutSecs = 0;
|
||||||
|
|
|
@ -50,7 +50,7 @@ import org.apache.hadoop.yarn.exceptions.YarnRemoteException;
|
||||||
import org.apache.hadoop.yarn.factories.RecordFactory;
|
import org.apache.hadoop.yarn.factories.RecordFactory;
|
||||||
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.ipc.YarnRPC;
|
import org.apache.hadoop.yarn.ipc.YarnRPC;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppState;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppState;
|
||||||
import org.apache.hadoop.yarn.service.Service.STATE;
|
import org.apache.hadoop.yarn.service.Service.STATE;
|
||||||
|
@ -85,7 +85,7 @@ public class TestApplicationACLs {
|
||||||
|
|
||||||
@BeforeClass
|
@BeforeClass
|
||||||
public static void setup() throws InterruptedException, IOException {
|
public static void setup() throws InterruptedException, IOException {
|
||||||
Store store = StoreFactory.getStore(conf);
|
RMStateStore store = StoreFactory.getStore(conf);
|
||||||
conf.setBoolean(YarnConfiguration.YARN_ACL_ENABLE, true);
|
conf.setBoolean(YarnConfiguration.YARN_ACL_ENABLE, true);
|
||||||
AccessControlList adminACL = new AccessControlList("");
|
AccessControlList adminACL = new AccessControlList("");
|
||||||
adminACL.addGroup(SUPER_GROUP);
|
adminACL.addGroup(SUPER_GROUP);
|
||||||
|
|
|
@ -182,7 +182,7 @@ public class TestClientRMService {
|
||||||
private RMAppImpl getRMApp(RMContext rmContext, YarnScheduler yarnScheduler,
|
private RMAppImpl getRMApp(RMContext rmContext, YarnScheduler yarnScheduler,
|
||||||
ApplicationId applicationId3, YarnConfiguration config, String queueName) {
|
ApplicationId applicationId3, YarnConfiguration config, String queueName) {
|
||||||
return new RMAppImpl(applicationId3, rmContext, config, null, null,
|
return new RMAppImpl(applicationId3, rmContext, config, null, null,
|
||||||
queueName, null, null, null, yarnScheduler, null, System
|
queueName, null, null, yarnScheduler, null, System
|
||||||
.currentTimeMillis());
|
.currentTimeMillis());
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -36,7 +36,6 @@ import org.apache.hadoop.yarn.api.records.NodeId;
|
||||||
import org.apache.hadoop.yarn.api.records.NodeState;
|
import org.apache.hadoop.yarn.api.records.NodeState;
|
||||||
import org.apache.hadoop.yarn.event.EventHandler;
|
import org.apache.hadoop.yarn.event.EventHandler;
|
||||||
import org.apache.hadoop.yarn.server.api.records.HeartbeatResponse;
|
import org.apache.hadoop.yarn.server.api.records.HeartbeatResponse;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resourcetracker.InlineDispatcher;
|
import org.apache.hadoop.yarn.server.resourcemanager.resourcetracker.InlineDispatcher;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeCleanContainerEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeCleanContainerEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEvent;
|
||||||
|
@ -79,7 +78,7 @@ public class TestRMNodeTransitions {
|
||||||
InlineDispatcher rmDispatcher = new InlineDispatcher();
|
InlineDispatcher rmDispatcher = new InlineDispatcher();
|
||||||
|
|
||||||
rmContext =
|
rmContext =
|
||||||
new RMContextImpl(new MemStore(), rmDispatcher, null, null, null,
|
new RMContextImpl(rmDispatcher, null, null, null,
|
||||||
mock(DelegationTokenRenewer.class), null, null, null);
|
mock(DelegationTokenRenewer.class), null, null, null);
|
||||||
scheduler = mock(YarnScheduler.class);
|
scheduler = mock(YarnScheduler.class);
|
||||||
doAnswer(
|
doAnswer(
|
||||||
|
|
|
@ -31,7 +31,7 @@ import org.apache.hadoop.yarn.api.records.NodeHealthStatus;
|
||||||
import org.apache.hadoop.yarn.api.records.Priority;
|
import org.apache.hadoop.yarn.api.records.Priority;
|
||||||
import org.apache.hadoop.yarn.api.records.Resource;
|
import org.apache.hadoop.yarn.api.records.Resource;
|
||||||
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
||||||
|
@ -47,7 +47,7 @@ public class TestResourceManager {
|
||||||
@Before
|
@Before
|
||||||
public void setUp() throws Exception {
|
public void setUp() throws Exception {
|
||||||
Configuration conf = new YarnConfiguration();
|
Configuration conf = new YarnConfiguration();
|
||||||
Store store = StoreFactory.getStore(conf);
|
RMStateStore store = StoreFactory.getStore(conf);
|
||||||
resourceManager = new ResourceManager(store);
|
resourceManager = new ResourceManager(store);
|
||||||
resourceManager.init(conf);
|
resourceManager.init(conf);
|
||||||
}
|
}
|
||||||
|
|
|
@ -32,7 +32,6 @@ import org.apache.hadoop.yarn.api.records.Container;
|
||||||
import org.apache.hadoop.yarn.api.records.ContainerId;
|
import org.apache.hadoop.yarn.api.records.ContainerId;
|
||||||
import org.apache.hadoop.yarn.api.records.FinalApplicationStatus;
|
import org.apache.hadoop.yarn.api.records.FinalApplicationStatus;
|
||||||
import org.apache.hadoop.yarn.api.records.YarnApplicationState;
|
import org.apache.hadoop.yarn.api.records.YarnApplicationState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppState;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppState;
|
||||||
|
@ -194,10 +193,6 @@ public abstract class MockAsm extends MockApps {
|
||||||
throw new UnsupportedOperationException("Not supported yet.");
|
throw new UnsupportedOperationException("Not supported yet.");
|
||||||
}
|
}
|
||||||
@Override
|
@Override
|
||||||
public ApplicationStore getApplicationStore() {
|
|
||||||
throw new UnsupportedOperationException("Not supported yet.");
|
|
||||||
}
|
|
||||||
@Override
|
|
||||||
public float getProgress() {
|
public float getProgress() {
|
||||||
throw new UnsupportedOperationException("Not supported yet.");
|
throw new UnsupportedOperationException("Not supported yet.");
|
||||||
}
|
}
|
||||||
|
|
|
@ -49,8 +49,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.Allocation;
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.Allocation;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.YarnScheduler;
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.YarnScheduler;
|
||||||
|
|
|
@ -53,9 +53,7 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore.RMState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store.RMState;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.Allocation;
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.Allocation;
|
||||||
|
|
|
@ -35,7 +35,6 @@ import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
||||||
import org.junit.After;
|
import org.junit.After;
|
||||||
|
|
|
@ -36,8 +36,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
||||||
import org.junit.Before;
|
import org.junit.Before;
|
||||||
import org.junit.Test;
|
import org.junit.Test;
|
||||||
|
|
|
@ -43,9 +43,7 @@ import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore.RMState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store.RMState;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
||||||
|
|
|
@ -39,7 +39,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager.NodeEventDispatcher;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager.NodeEventDispatcher;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceTrackerService;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceTrackerService;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.SchedulerEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.SchedulerEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.security.RMContainerTokenSecretManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.security.RMContainerTokenSecretManager;
|
||||||
|
@ -70,7 +69,7 @@ public class TestNMExpiry {
|
||||||
Configuration conf = new Configuration();
|
Configuration conf = new Configuration();
|
||||||
// Dispatcher that processes events inline
|
// Dispatcher that processes events inline
|
||||||
Dispatcher dispatcher = new InlineDispatcher();
|
Dispatcher dispatcher = new InlineDispatcher();
|
||||||
RMContext context = new RMContextImpl(new MemStore(), dispatcher, null,
|
RMContext context = new RMContextImpl(dispatcher, null,
|
||||||
null, null, null, null, null, null);
|
null, null, null, null, null, null);
|
||||||
dispatcher.register(SchedulerEventType.class,
|
dispatcher.register(SchedulerEventType.class,
|
||||||
new InlineDispatcher.EmptyEventHandler());
|
new InlineDispatcher.EmptyEventHandler());
|
||||||
|
|
|
@ -39,7 +39,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceTrackerService;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceTrackerService;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNodeEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.SchedulerEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.SchedulerEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.security.RMContainerTokenSecretManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.security.RMContainerTokenSecretManager;
|
||||||
|
@ -66,7 +65,7 @@ public class TestRMNMRPCResponseId {
|
||||||
}
|
}
|
||||||
});
|
});
|
||||||
RMContext context =
|
RMContext context =
|
||||||
new RMContextImpl(new MemStore(), dispatcher, null, null, null, null,
|
new RMContextImpl(dispatcher, null, null, null, null,
|
||||||
null, null, null);
|
null, null, null);
|
||||||
dispatcher.register(RMNodeEventType.class,
|
dispatcher.register(RMNodeEventType.class,
|
||||||
new ResourceManager.NodeEventDispatcher(context));
|
new ResourceManager.NodeEventDispatcher(context));
|
||||||
|
|
|
@ -28,7 +28,6 @@ import org.apache.hadoop.yarn.api.records.FinalApplicationStatus;
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
import org.apache.hadoop.yarn.api.records.ApplicationId;
|
||||||
import org.apache.hadoop.yarn.api.records.ApplicationReport;
|
import org.apache.hadoop.yarn.api.records.ApplicationReport;
|
||||||
import org.apache.hadoop.yarn.MockApps;
|
import org.apache.hadoop.yarn.MockApps;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
||||||
|
|
||||||
|
@ -137,11 +136,6 @@ public class MockRMApp implements RMApp {
|
||||||
throw new UnsupportedOperationException("Not supported yet.");
|
throw new UnsupportedOperationException("Not supported yet.");
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
|
||||||
public ApplicationStore getApplicationStore() {
|
|
||||||
throw new UnsupportedOperationException("Not supported yet.");
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public long getFinishTime() {
|
public long getFinishTime() {
|
||||||
return finish;
|
return finish;
|
||||||
|
|
|
@ -40,8 +40,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMAppManagerEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMAppManagerEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMAppManagerEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.ApplicationsStore.ApplicationStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.AMLivelinessMonitor;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttempt;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEvent;
|
||||||
|
@ -140,7 +138,7 @@ public class TestRMAppTransitions {
|
||||||
AMLivelinessMonitor amLivelinessMonitor = mock(AMLivelinessMonitor.class);
|
AMLivelinessMonitor amLivelinessMonitor = mock(AMLivelinessMonitor.class);
|
||||||
AMLivelinessMonitor amFinishingMonitor = mock(AMLivelinessMonitor.class);
|
AMLivelinessMonitor amFinishingMonitor = mock(AMLivelinessMonitor.class);
|
||||||
this.rmContext =
|
this.rmContext =
|
||||||
new RMContextImpl(new MemStore(), rmDispatcher,
|
new RMContextImpl(rmDispatcher,
|
||||||
containerAllocationExpirer, amLivelinessMonitor, amFinishingMonitor,
|
containerAllocationExpirer, amLivelinessMonitor, amFinishingMonitor,
|
||||||
null, new ApplicationTokenSecretManager(conf),
|
null, new ApplicationTokenSecretManager(conf),
|
||||||
new RMContainerTokenSecretManager(conf),
|
new RMContainerTokenSecretManager(conf),
|
||||||
|
@ -171,7 +169,6 @@ public class TestRMAppTransitions {
|
||||||
// ensure max retries set to known value
|
// ensure max retries set to known value
|
||||||
conf.setInt(YarnConfiguration.RM_AM_MAX_RETRIES, maxRetries);
|
conf.setInt(YarnConfiguration.RM_AM_MAX_RETRIES, maxRetries);
|
||||||
String clientTokenStr = "bogusstring";
|
String clientTokenStr = "bogusstring";
|
||||||
ApplicationStore appStore = mock(ApplicationStore.class);
|
|
||||||
YarnScheduler scheduler = mock(YarnScheduler.class);
|
YarnScheduler scheduler = mock(YarnScheduler.class);
|
||||||
ApplicationMasterService masterService =
|
ApplicationMasterService masterService =
|
||||||
new ApplicationMasterService(rmContext, scheduler);
|
new ApplicationMasterService(rmContext, scheduler);
|
||||||
|
@ -183,7 +180,7 @@ public class TestRMAppTransitions {
|
||||||
RMApp application = new RMAppImpl(applicationId, rmContext,
|
RMApp application = new RMAppImpl(applicationId, rmContext,
|
||||||
conf, name, user,
|
conf, name, user,
|
||||||
queue, submissionContext, clientTokenStr,
|
queue, submissionContext, clientTokenStr,
|
||||||
appStore, scheduler,
|
scheduler,
|
||||||
masterService, System.currentTimeMillis());
|
masterService, System.currentTimeMillis());
|
||||||
|
|
||||||
testAppStartState(applicationId, user, name, queue, application);
|
testAppStartState(applicationId, user, name, queue, application);
|
||||||
|
|
|
@ -53,7 +53,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEvent;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.AMLauncherEventType;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.ApplicationMasterLauncher;
|
import org.apache.hadoop.yarn.server.resourcemanager.amlauncher.ApplicationMasterLauncher;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resourcetracker.InlineDispatcher;
|
import org.apache.hadoop.yarn.server.resourcemanager.resourcetracker.InlineDispatcher;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppEvent;
|
||||||
|
@ -162,7 +161,7 @@ public class TestRMAppAttemptTransitions {
|
||||||
amFinishingMonitor = mock(AMLivelinessMonitor.class);
|
amFinishingMonitor = mock(AMLivelinessMonitor.class);
|
||||||
Configuration conf = new Configuration();
|
Configuration conf = new Configuration();
|
||||||
rmContext =
|
rmContext =
|
||||||
new RMContextImpl(new MemStore(), rmDispatcher,
|
new RMContextImpl(rmDispatcher,
|
||||||
containerAllocationExpirer, amLivelinessMonitor, amFinishingMonitor,
|
containerAllocationExpirer, amLivelinessMonitor, amFinishingMonitor,
|
||||||
null, new ApplicationTokenSecretManager(conf),
|
null, new ApplicationTokenSecretManager(conf),
|
||||||
new RMContainerTokenSecretManager(conf),
|
new RMContainerTokenSecretManager(conf),
|
||||||
|
|
|
@ -479,7 +479,7 @@ public class TestApplicationLimits {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0_0 =
|
FiCaSchedulerApp app_0_0 =
|
||||||
spy(new FiCaSchedulerApp(appAttemptId_0_0, user_0, queue,
|
spy(new FiCaSchedulerApp(appAttemptId_0_0, user_0, queue,
|
||||||
queue.getActiveUsersManager(), rmContext, null));
|
queue.getActiveUsersManager(), rmContext));
|
||||||
queue.submitApplication(app_0_0, user_0, A);
|
queue.submitApplication(app_0_0, user_0, A);
|
||||||
|
|
||||||
List<ResourceRequest> app_0_0_requests = new ArrayList<ResourceRequest>();
|
List<ResourceRequest> app_0_0_requests = new ArrayList<ResourceRequest>();
|
||||||
|
@ -498,7 +498,7 @@ public class TestApplicationLimits {
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_0_1 =
|
FiCaSchedulerApp app_0_1 =
|
||||||
spy(new FiCaSchedulerApp(appAttemptId_0_1, user_0, queue,
|
spy(new FiCaSchedulerApp(appAttemptId_0_1, user_0, queue,
|
||||||
queue.getActiveUsersManager(), rmContext, null));
|
queue.getActiveUsersManager(), rmContext));
|
||||||
queue.submitApplication(app_0_1, user_0, A);
|
queue.submitApplication(app_0_1, user_0, A);
|
||||||
|
|
||||||
List<ResourceRequest> app_0_1_requests = new ArrayList<ResourceRequest>();
|
List<ResourceRequest> app_0_1_requests = new ArrayList<ResourceRequest>();
|
||||||
|
@ -517,7 +517,7 @@ public class TestApplicationLimits {
|
||||||
TestUtils.getMockApplicationAttemptId(2, 0);
|
TestUtils.getMockApplicationAttemptId(2, 0);
|
||||||
FiCaSchedulerApp app_1_0 =
|
FiCaSchedulerApp app_1_0 =
|
||||||
spy(new FiCaSchedulerApp(appAttemptId_1_0, user_1, queue,
|
spy(new FiCaSchedulerApp(appAttemptId_1_0, user_1, queue,
|
||||||
queue.getActiveUsersManager(), rmContext, null));
|
queue.getActiveUsersManager(), rmContext));
|
||||||
queue.submitApplication(app_1_0, user_1, A);
|
queue.submitApplication(app_1_0, user_1, A);
|
||||||
|
|
||||||
List<ResourceRequest> app_1_0_requests = new ArrayList<ResourceRequest>();
|
List<ResourceRequest> app_1_0_requests = new ArrayList<ResourceRequest>();
|
||||||
|
|
|
@ -40,7 +40,7 @@ import org.apache.hadoop.yarn.server.resourcemanager.MockNodes;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.Task;
|
import org.apache.hadoop.yarn.server.resourcemanager.Task;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
||||||
|
@ -76,7 +76,7 @@ public class TestCapacityScheduler {
|
||||||
|
|
||||||
@Before
|
@Before
|
||||||
public void setUp() throws Exception {
|
public void setUp() throws Exception {
|
||||||
Store store = StoreFactory.getStore(new Configuration());
|
RMStateStore store = StoreFactory.getStore(new Configuration());
|
||||||
resourceManager = new ResourceManager(store);
|
resourceManager = new ResourceManager(store);
|
||||||
CapacitySchedulerConfiguration csConf
|
CapacitySchedulerConfiguration csConf
|
||||||
= new CapacitySchedulerConfiguration();
|
= new CapacitySchedulerConfiguration();
|
||||||
|
@ -251,7 +251,7 @@ public class TestCapacityScheduler {
|
||||||
CapacitySchedulerConfiguration conf = new CapacitySchedulerConfiguration();
|
CapacitySchedulerConfiguration conf = new CapacitySchedulerConfiguration();
|
||||||
setupQueueConfiguration(conf);
|
setupQueueConfiguration(conf);
|
||||||
cs.setConf(new YarnConfiguration());
|
cs.setConf(new YarnConfiguration());
|
||||||
cs.reinitialize(conf, new RMContextImpl(null, null, null, null, null, null,
|
cs.reinitialize(conf, new RMContextImpl(null, null, null, null, null,
|
||||||
null, new RMContainerTokenSecretManager(conf),
|
null, new RMContainerTokenSecretManager(conf),
|
||||||
new ClientToAMTokenSecretManagerInRM()));
|
new ClientToAMTokenSecretManagerInRM()));
|
||||||
checkQueueCapacities(cs, A_CAPACITY, B_CAPACITY);
|
checkQueueCapacities(cs, A_CAPACITY, B_CAPACITY);
|
||||||
|
@ -349,7 +349,7 @@ public class TestCapacityScheduler {
|
||||||
conf.setCapacity(CapacitySchedulerConfiguration.ROOT + ".a.a1.b1", 100.0f);
|
conf.setCapacity(CapacitySchedulerConfiguration.ROOT + ".a.a1.b1", 100.0f);
|
||||||
conf.setUserLimitFactor(CapacitySchedulerConfiguration.ROOT + ".a.a1.b1", 100.0f);
|
conf.setUserLimitFactor(CapacitySchedulerConfiguration.ROOT + ".a.a1.b1", 100.0f);
|
||||||
|
|
||||||
cs.reinitialize(conf, new RMContextImpl(null, null, null, null, null, null,
|
cs.reinitialize(conf, new RMContextImpl(null, null, null, null, null,
|
||||||
null, new RMContainerTokenSecretManager(conf),
|
null, new RMContainerTokenSecretManager(conf),
|
||||||
new ClientToAMTokenSecretManagerInRM()));
|
new ClientToAMTokenSecretManagerInRM()));
|
||||||
}
|
}
|
||||||
|
@ -361,7 +361,7 @@ public class TestCapacityScheduler {
|
||||||
setupQueueConfiguration(csConf);
|
setupQueueConfiguration(csConf);
|
||||||
CapacityScheduler cs = new CapacityScheduler();
|
CapacityScheduler cs = new CapacityScheduler();
|
||||||
cs.setConf(new YarnConfiguration());
|
cs.setConf(new YarnConfiguration());
|
||||||
cs.reinitialize(csConf, new RMContextImpl(null, null, null, null, null,
|
cs.reinitialize(csConf, new RMContextImpl(null, null, null, null,
|
||||||
null, null, new RMContainerTokenSecretManager(csConf),
|
null, null, new RMContainerTokenSecretManager(csConf),
|
||||||
new ClientToAMTokenSecretManagerInRM()));
|
new ClientToAMTokenSecretManagerInRM()));
|
||||||
|
|
||||||
|
@ -387,7 +387,7 @@ public class TestCapacityScheduler {
|
||||||
CapacitySchedulerConfiguration conf = new CapacitySchedulerConfiguration();
|
CapacitySchedulerConfiguration conf = new CapacitySchedulerConfiguration();
|
||||||
setupQueueConfiguration(conf);
|
setupQueueConfiguration(conf);
|
||||||
cs.setConf(new YarnConfiguration());
|
cs.setConf(new YarnConfiguration());
|
||||||
cs.reinitialize(conf, new RMContextImpl(null, null, null, null, null, null,
|
cs.reinitialize(conf, new RMContextImpl(null, null, null, null, null,
|
||||||
null, new RMContainerTokenSecretManager(conf),
|
null, new RMContainerTokenSecretManager(conf),
|
||||||
new ClientToAMTokenSecretManagerInRM()));
|
new ClientToAMTokenSecretManagerInRM()));
|
||||||
checkQueueCapacities(cs, A_CAPACITY, B_CAPACITY);
|
checkQueueCapacities(cs, A_CAPACITY, B_CAPACITY);
|
||||||
|
|
|
@ -250,14 +250,14 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_0, user_0, B);
|
a.submitApplication(app_0, user_0, B);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_1 =
|
final ApplicationAttemptId appAttemptId_1 =
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_1 =
|
FiCaSchedulerApp app_1 =
|
||||||
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_1, user_0, B); // same user
|
a.submitApplication(app_1, user_0, B); // same user
|
||||||
|
|
||||||
|
|
||||||
|
@ -295,14 +295,14 @@ public class TestLeafQueue {
|
||||||
final ApplicationAttemptId appAttemptId_0 = TestUtils
|
final ApplicationAttemptId appAttemptId_0 = TestUtils
|
||||||
.getMockApplicationAttemptId(0, 1);
|
.getMockApplicationAttemptId(0, 1);
|
||||||
FiCaSchedulerApp app_0 = new FiCaSchedulerApp(appAttemptId_0, user_d, d, null,
|
FiCaSchedulerApp app_0 = new FiCaSchedulerApp(appAttemptId_0, user_d, d, null,
|
||||||
rmContext, null);
|
rmContext);
|
||||||
d.submitApplication(app_0, user_d, D);
|
d.submitApplication(app_0, user_d, D);
|
||||||
|
|
||||||
// Attempt the same application again
|
// Attempt the same application again
|
||||||
final ApplicationAttemptId appAttemptId_1 = TestUtils
|
final ApplicationAttemptId appAttemptId_1 = TestUtils
|
||||||
.getMockApplicationAttemptId(0, 2);
|
.getMockApplicationAttemptId(0, 2);
|
||||||
FiCaSchedulerApp app_1 = new FiCaSchedulerApp(appAttemptId_1, user_d, d, null,
|
FiCaSchedulerApp app_1 = new FiCaSchedulerApp(appAttemptId_1, user_d, d, null,
|
||||||
rmContext, null);
|
rmContext);
|
||||||
d.submitApplication(app_1, user_d, D); // same user
|
d.submitApplication(app_1, user_d, D); // same user
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -320,7 +320,7 @@ public class TestLeafQueue {
|
||||||
final ApplicationAttemptId appAttemptId_0 = TestUtils
|
final ApplicationAttemptId appAttemptId_0 = TestUtils
|
||||||
.getMockApplicationAttemptId(0, 1);
|
.getMockApplicationAttemptId(0, 1);
|
||||||
FiCaSchedulerApp app_0 = new FiCaSchedulerApp(appAttemptId_0, user_0, a, null,
|
FiCaSchedulerApp app_0 = new FiCaSchedulerApp(appAttemptId_0, user_0, a, null,
|
||||||
rmContext, null);
|
rmContext);
|
||||||
a.submitApplication(app_0, user_0, B);
|
a.submitApplication(app_0, user_0, B);
|
||||||
|
|
||||||
when(cs.getApplication(appAttemptId_0)).thenReturn(app_0);
|
when(cs.getApplication(appAttemptId_0)).thenReturn(app_0);
|
||||||
|
@ -335,7 +335,7 @@ public class TestLeafQueue {
|
||||||
final ApplicationAttemptId appAttemptId_1 = TestUtils
|
final ApplicationAttemptId appAttemptId_1 = TestUtils
|
||||||
.getMockApplicationAttemptId(0, 2);
|
.getMockApplicationAttemptId(0, 2);
|
||||||
FiCaSchedulerApp app_1 = new FiCaSchedulerApp(appAttemptId_1, user_0, a, null,
|
FiCaSchedulerApp app_1 = new FiCaSchedulerApp(appAttemptId_1, user_0, a, null,
|
||||||
rmContext, null);
|
rmContext);
|
||||||
a.submitApplication(app_1, user_0, B); // same user
|
a.submitApplication(app_1, user_0, B); // same user
|
||||||
|
|
||||||
assertEquals(1, a.getMetrics().getAppsSubmitted());
|
assertEquals(1, a.getMetrics().getAppsSubmitted());
|
||||||
|
@ -371,14 +371,14 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_1 =
|
final ApplicationAttemptId appAttemptId_1 =
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_1 =
|
FiCaSchedulerApp app_1 =
|
||||||
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_1, user_0, A); // same user
|
a.submitApplication(app_1, user_0, A); // same user
|
||||||
|
|
||||||
|
|
||||||
|
@ -495,21 +495,21 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_1 =
|
final ApplicationAttemptId appAttemptId_1 =
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_1 =
|
FiCaSchedulerApp app_1 =
|
||||||
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_1, user_0, A); // same user
|
a.submitApplication(app_1, user_0, A); // same user
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_2 =
|
final ApplicationAttemptId appAttemptId_2 =
|
||||||
TestUtils.getMockApplicationAttemptId(2, 0);
|
TestUtils.getMockApplicationAttemptId(2, 0);
|
||||||
FiCaSchedulerApp app_2 =
|
FiCaSchedulerApp app_2 =
|
||||||
new FiCaSchedulerApp(appAttemptId_2, user_1, a,
|
new FiCaSchedulerApp(appAttemptId_2, user_1, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_2, user_1, A);
|
a.submitApplication(app_2, user_1, A);
|
||||||
|
|
||||||
// Setup some nodes
|
// Setup some nodes
|
||||||
|
@ -588,21 +588,21 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_1 =
|
final ApplicationAttemptId appAttemptId_1 =
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_1 =
|
FiCaSchedulerApp app_1 =
|
||||||
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_1, user_0, A); // same user
|
a.submitApplication(app_1, user_0, A); // same user
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_2 =
|
final ApplicationAttemptId appAttemptId_2 =
|
||||||
TestUtils.getMockApplicationAttemptId(2, 0);
|
TestUtils.getMockApplicationAttemptId(2, 0);
|
||||||
FiCaSchedulerApp app_2 =
|
FiCaSchedulerApp app_2 =
|
||||||
new FiCaSchedulerApp(appAttemptId_2, user_1, a,
|
new FiCaSchedulerApp(appAttemptId_2, user_1, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_2, user_1, A);
|
a.submitApplication(app_2, user_1, A);
|
||||||
|
|
||||||
// Setup some nodes
|
// Setup some nodes
|
||||||
|
@ -699,28 +699,28 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_1 =
|
final ApplicationAttemptId appAttemptId_1 =
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_1 =
|
FiCaSchedulerApp app_1 =
|
||||||
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_1, user_0, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_1, user_0, A); // same user
|
a.submitApplication(app_1, user_0, A); // same user
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_2 =
|
final ApplicationAttemptId appAttemptId_2 =
|
||||||
TestUtils.getMockApplicationAttemptId(2, 0);
|
TestUtils.getMockApplicationAttemptId(2, 0);
|
||||||
FiCaSchedulerApp app_2 =
|
FiCaSchedulerApp app_2 =
|
||||||
new FiCaSchedulerApp(appAttemptId_2, user_1, a,
|
new FiCaSchedulerApp(appAttemptId_2, user_1, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_2, user_1, A);
|
a.submitApplication(app_2, user_1, A);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_3 =
|
final ApplicationAttemptId appAttemptId_3 =
|
||||||
TestUtils.getMockApplicationAttemptId(3, 0);
|
TestUtils.getMockApplicationAttemptId(3, 0);
|
||||||
FiCaSchedulerApp app_3 =
|
FiCaSchedulerApp app_3 =
|
||||||
new FiCaSchedulerApp(appAttemptId_3, user_2, a,
|
new FiCaSchedulerApp(appAttemptId_3, user_2, a,
|
||||||
a.getActiveUsersManager(), rmContext, null);
|
a.getActiveUsersManager(), rmContext);
|
||||||
a.submitApplication(app_3, user_2, A);
|
a.submitApplication(app_3, user_2, A);
|
||||||
|
|
||||||
// Setup some nodes
|
// Setup some nodes
|
||||||
|
@ -874,14 +874,14 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_1 =
|
final ApplicationAttemptId appAttemptId_1 =
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_1 =
|
FiCaSchedulerApp app_1 =
|
||||||
new FiCaSchedulerApp(appAttemptId_1, user_1, a,
|
new FiCaSchedulerApp(appAttemptId_1, user_1, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_1, user_1, A);
|
a.submitApplication(app_1, user_1, A);
|
||||||
|
|
||||||
// Setup some nodes
|
// Setup some nodes
|
||||||
|
@ -973,14 +973,14 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_1 =
|
final ApplicationAttemptId appAttemptId_1 =
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_1 =
|
FiCaSchedulerApp app_1 =
|
||||||
new FiCaSchedulerApp(appAttemptId_1, user_1, a,
|
new FiCaSchedulerApp(appAttemptId_1, user_1, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_1, user_1, A);
|
a.submitApplication(app_1, user_1, A);
|
||||||
|
|
||||||
// Setup some nodes
|
// Setup some nodes
|
||||||
|
@ -1072,14 +1072,14 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
final ApplicationAttemptId appAttemptId_1 =
|
final ApplicationAttemptId appAttemptId_1 =
|
||||||
TestUtils.getMockApplicationAttemptId(1, 0);
|
TestUtils.getMockApplicationAttemptId(1, 0);
|
||||||
FiCaSchedulerApp app_1 =
|
FiCaSchedulerApp app_1 =
|
||||||
new FiCaSchedulerApp(appAttemptId_1, user_1, a,
|
new FiCaSchedulerApp(appAttemptId_1, user_1, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null);
|
mock(ActiveUsersManager.class), rmContext);
|
||||||
a.submitApplication(app_1, user_1, A);
|
a.submitApplication(app_1, user_1, A);
|
||||||
|
|
||||||
// Setup some nodes
|
// Setup some nodes
|
||||||
|
@ -1187,7 +1187,7 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
spy(new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
spy(new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null));
|
mock(ActiveUsersManager.class), rmContext));
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
// Setup some nodes and racks
|
// Setup some nodes and racks
|
||||||
|
@ -1327,7 +1327,7 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
spy(new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
spy(new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null));
|
mock(ActiveUsersManager.class), rmContext));
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
// Setup some nodes and racks
|
// Setup some nodes and racks
|
||||||
|
@ -1457,7 +1457,7 @@ public class TestLeafQueue {
|
||||||
TestUtils.getMockApplicationAttemptId(0, 0);
|
TestUtils.getMockApplicationAttemptId(0, 0);
|
||||||
FiCaSchedulerApp app_0 =
|
FiCaSchedulerApp app_0 =
|
||||||
spy(new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
spy(new FiCaSchedulerApp(appAttemptId_0, user_0, a,
|
||||||
mock(ActiveUsersManager.class), rmContext, null));
|
mock(ActiveUsersManager.class), rmContext));
|
||||||
a.submitApplication(app_0, user_0, A);
|
a.submitApplication(app_0, user_0, A);
|
||||||
|
|
||||||
// Setup some nodes and racks
|
// Setup some nodes and racks
|
||||||
|
|
|
@ -43,7 +43,7 @@ public class TestQueueParsing {
|
||||||
|
|
||||||
CapacityScheduler capacityScheduler = new CapacityScheduler();
|
CapacityScheduler capacityScheduler = new CapacityScheduler();
|
||||||
capacityScheduler.setConf(conf);
|
capacityScheduler.setConf(conf);
|
||||||
capacityScheduler.reinitialize(conf, new RMContextImpl(null, null, null,
|
capacityScheduler.reinitialize(conf, new RMContextImpl(null, null,
|
||||||
null, null, null, null, new RMContainerTokenSecretManager(conf),
|
null, null, null, null, new RMContainerTokenSecretManager(conf),
|
||||||
new ClientToAMTokenSecretManagerInRM()));
|
new ClientToAMTokenSecretManagerInRM()));
|
||||||
|
|
||||||
|
|
|
@ -83,7 +83,7 @@ public class TestUtils {
|
||||||
|
|
||||||
Configuration conf = new Configuration();
|
Configuration conf = new Configuration();
|
||||||
RMContext rmContext =
|
RMContext rmContext =
|
||||||
new RMContextImpl(null, nullDispatcher, cae, null, null, null,
|
new RMContextImpl(nullDispatcher, cae, null, null, null,
|
||||||
new ApplicationTokenSecretManager(conf),
|
new ApplicationTokenSecretManager(conf),
|
||||||
new RMContainerTokenSecretManager(conf),
|
new RMContainerTokenSecretManager(conf),
|
||||||
new ClientToAMTokenSecretManagerInRM());
|
new ClientToAMTokenSecretManagerInRM());
|
||||||
|
|
|
@ -53,7 +53,7 @@ public class TestFSSchedulerApp {
|
||||||
|
|
||||||
ApplicationAttemptId applicationAttemptId = createAppAttemptId(1, 1);
|
ApplicationAttemptId applicationAttemptId = createAppAttemptId(1, 1);
|
||||||
FSSchedulerApp schedulerApp =
|
FSSchedulerApp schedulerApp =
|
||||||
new FSSchedulerApp(applicationAttemptId, "user1", queue , null, null, null);
|
new FSSchedulerApp(applicationAttemptId, "user1", queue , null, null);
|
||||||
|
|
||||||
// Default level should be node-local
|
// Default level should be node-local
|
||||||
assertEquals(NodeType.NODE_LOCAL, schedulerApp.getAllowedLocalityLevel(
|
assertEquals(NodeType.NODE_LOCAL, schedulerApp.getAllowedLocalityLevel(
|
||||||
|
@ -111,7 +111,7 @@ public class TestFSSchedulerApp {
|
||||||
|
|
||||||
ApplicationAttemptId applicationAttemptId = createAppAttemptId(1, 1);
|
ApplicationAttemptId applicationAttemptId = createAppAttemptId(1, 1);
|
||||||
FSSchedulerApp schedulerApp =
|
FSSchedulerApp schedulerApp =
|
||||||
new FSSchedulerApp(applicationAttemptId, "user1", queue , null, null, null);
|
new FSSchedulerApp(applicationAttemptId, "user1", queue , null, null);
|
||||||
assertEquals(NodeType.OFF_SWITCH, schedulerApp.getAllowedLocalityLevel(
|
assertEquals(NodeType.OFF_SWITCH, schedulerApp.getAllowedLocalityLevel(
|
||||||
prio, 10, -1.0, -1.0));
|
prio, 10, -1.0, -1.0));
|
||||||
}
|
}
|
||||||
|
|
|
@ -47,7 +47,7 @@ import org.apache.hadoop.yarn.factories.RecordFactory;
|
||||||
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.MockNodes;
|
import org.apache.hadoop.yarn.server.resourcemanager.MockNodes;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptState;
|
||||||
|
@ -97,7 +97,7 @@ public class TestFairScheduler {
|
||||||
Configuration conf = createConfiguration();
|
Configuration conf = createConfiguration();
|
||||||
// All tests assume only one assignment per node update
|
// All tests assume only one assignment per node update
|
||||||
conf.set(FairSchedulerConfiguration.ASSIGN_MULTIPLE, "false");
|
conf.set(FairSchedulerConfiguration.ASSIGN_MULTIPLE, "false");
|
||||||
Store store = StoreFactory.getStore(conf);
|
RMStateStore store = StoreFactory.getStore(conf);
|
||||||
resourceManager = new ResourceManager(store);
|
resourceManager = new ResourceManager(store);
|
||||||
resourceManager.init(conf);
|
resourceManager.init(conf);
|
||||||
((AsyncDispatcher)resourceManager.getRMContext().getDispatcher()).start();
|
((AsyncDispatcher)resourceManager.getRMContext().getDispatcher()).start();
|
||||||
|
|
|
@ -27,7 +27,7 @@ import org.apache.hadoop.conf.Configuration;
|
||||||
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
||||||
import org.apache.hadoop.yarn.event.AsyncDispatcher;
|
import org.apache.hadoop.yarn.event.AsyncDispatcher;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler;
|
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler;
|
||||||
import org.junit.After;
|
import org.junit.After;
|
||||||
|
@ -50,7 +50,7 @@ public class TestFairSchedulerEventLog {
|
||||||
|
|
||||||
// All tests assume only one assignment per node update
|
// All tests assume only one assignment per node update
|
||||||
conf.set(FairSchedulerConfiguration.ASSIGN_MULTIPLE, "false");
|
conf.set(FairSchedulerConfiguration.ASSIGN_MULTIPLE, "false");
|
||||||
Store store = StoreFactory.getStore(conf);
|
RMStateStore store = StoreFactory.getStore(conf);
|
||||||
resourceManager = new ResourceManager(store);
|
resourceManager = new ResourceManager(store);
|
||||||
resourceManager.init(conf);
|
resourceManager.init(conf);
|
||||||
((AsyncDispatcher)resourceManager.getRMContext().getDispatcher()).start();
|
((AsyncDispatcher)resourceManager.getRMContext().getDispatcher()).start();
|
||||||
|
|
|
@ -38,7 +38,7 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.Task;
|
import org.apache.hadoop.yarn.server.resourcemanager.Task;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
import org.apache.hadoop.yarn.server.resourcemanager.resource.Resources;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.resourcetracker.InlineDispatcher;
|
import org.apache.hadoop.yarn.server.resourcemanager.resourcetracker.InlineDispatcher;
|
||||||
|
@ -59,7 +59,7 @@ public class TestFifoScheduler {
|
||||||
|
|
||||||
@Before
|
@Before
|
||||||
public void setUp() throws Exception {
|
public void setUp() throws Exception {
|
||||||
Store store = StoreFactory.getStore(new Configuration());
|
RMStateStore store = StoreFactory.getStore(new Configuration());
|
||||||
resourceManager = new ResourceManager(store);
|
resourceManager = new ResourceManager(store);
|
||||||
Configuration conf = new Configuration();
|
Configuration conf = new Configuration();
|
||||||
conf.setClass(YarnConfiguration.RM_SCHEDULER,
|
conf.setClass(YarnConfiguration.RM_SCHEDULER,
|
||||||
|
@ -91,7 +91,7 @@ public class TestFifoScheduler {
|
||||||
@Test
|
@Test
|
||||||
public void testAppAttemptMetrics() throws Exception {
|
public void testAppAttemptMetrics() throws Exception {
|
||||||
AsyncDispatcher dispatcher = new InlineDispatcher();
|
AsyncDispatcher dispatcher = new InlineDispatcher();
|
||||||
RMContext rmContext = new RMContextImpl(null, dispatcher, null,
|
RMContext rmContext = new RMContextImpl(dispatcher, null,
|
||||||
null, null, null, null, null, null);
|
null, null, null, null, null, null);
|
||||||
|
|
||||||
FifoScheduler schedular = new FifoScheduler();
|
FifoScheduler schedular = new FifoScheduler();
|
||||||
|
|
|
@ -38,7 +38,6 @@ import org.apache.hadoop.yarn.server.resourcemanager.RMContext;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
import org.apache.hadoop.yarn.server.resourcemanager.RMContextImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.applicationsmanager.MockAsm;
|
import org.apache.hadoop.yarn.server.resourcemanager.applicationsmanager.MockAsm;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.MemStore;
|
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMApp;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppState;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.RMAppState;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
import org.apache.hadoop.yarn.server.resourcemanager.rmnode.RMNode;
|
||||||
|
@ -160,7 +159,7 @@ public class TestRMWebApp {
|
||||||
for (RMNode node : deactivatedNodes) {
|
for (RMNode node : deactivatedNodes) {
|
||||||
deactivatedNodesMap.put(node.getHostName(), node);
|
deactivatedNodesMap.put(node.getHostName(), node);
|
||||||
}
|
}
|
||||||
return new RMContextImpl(new MemStore(), null, null, null, null,
|
return new RMContextImpl(null, null, null, null,
|
||||||
null, null, null, null) {
|
null, null, null, null) {
|
||||||
@Override
|
@Override
|
||||||
public ConcurrentMap<ApplicationId, RMApp> getRMApps() {
|
public ConcurrentMap<ApplicationId, RMApp> getRMApps() {
|
||||||
|
@ -201,7 +200,7 @@ public class TestRMWebApp {
|
||||||
|
|
||||||
CapacityScheduler cs = new CapacityScheduler();
|
CapacityScheduler cs = new CapacityScheduler();
|
||||||
cs.setConf(new YarnConfiguration());
|
cs.setConf(new YarnConfiguration());
|
||||||
cs.reinitialize(conf, new RMContextImpl(null, null, null, null, null, null,
|
cs.reinitialize(conf, new RMContextImpl(null, null, null, null, null,
|
||||||
null, new RMContainerTokenSecretManager(conf),
|
null, new RMContainerTokenSecretManager(conf),
|
||||||
new ClientToAMTokenSecretManagerInRM()));
|
new ClientToAMTokenSecretManagerInRM()));
|
||||||
return cs;
|
return cs;
|
||||||
|
|
|
@ -48,7 +48,7 @@ import org.apache.hadoop.yarn.server.nodemanager.NodeStatusUpdater;
|
||||||
import org.apache.hadoop.yarn.server.nodemanager.NodeStatusUpdaterImpl;
|
import org.apache.hadoop.yarn.server.nodemanager.NodeStatusUpdaterImpl;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.ResourceTrackerService;
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceTrackerService;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.Store;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.RMStateStore;
|
||||||
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
import org.apache.hadoop.yarn.server.resourcemanager.recovery.StoreFactory;
|
||||||
import org.apache.hadoop.yarn.service.AbstractService;
|
import org.apache.hadoop.yarn.service.AbstractService;
|
||||||
import org.apache.hadoop.yarn.service.CompositeService;
|
import org.apache.hadoop.yarn.service.CompositeService;
|
||||||
|
@ -154,7 +154,7 @@ public class MiniYARNCluster extends CompositeService {
|
||||||
getConfig().set(YarnConfiguration.RM_WEBAPP_ADDRESS,
|
getConfig().set(YarnConfiguration.RM_WEBAPP_ADDRESS,
|
||||||
MiniYARNCluster.getHostname() + ":0");
|
MiniYARNCluster.getHostname() + ":0");
|
||||||
}
|
}
|
||||||
Store store = StoreFactory.getStore(getConfig());
|
RMStateStore store = StoreFactory.getStore(getConfig());
|
||||||
resourceManager = new ResourceManager(store) {
|
resourceManager = new ResourceManager(store) {
|
||||||
@Override
|
@Override
|
||||||
protected void doSecureLogin() throws IOException {
|
protected void doSecureLogin() throws IOException {
|
||||||
|
|
Loading…
Reference in New Issue