|
@@ -35,21 +35,23 @@ import org.apache.hadoop.fs.FileContext;
|
|
|
import org.apache.hadoop.fs.Path;
|
|
|
import org.apache.hadoop.ha.HAServiceProtocol;
|
|
|
import org.apache.hadoop.metrics2.lib.DefaultMetricsSystem;
|
|
|
+import org.apache.hadoop.security.token.Token;
|
|
|
import org.apache.hadoop.service.AbstractService;
|
|
|
import org.apache.hadoop.service.CompositeService;
|
|
|
import org.apache.hadoop.util.Shell;
|
|
|
import org.apache.hadoop.util.Shell.ShellCommandExecutor;
|
|
|
import org.apache.hadoop.yarn.api.protocolrecords.GetClusterMetricsRequest;
|
|
|
import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
|
|
|
-import org.apache.hadoop.yarn.api.records.Resource;
|
|
|
import org.apache.hadoop.yarn.conf.HAUtil;
|
|
|
import org.apache.hadoop.yarn.conf.YarnConfiguration;
|
|
|
+import org.apache.hadoop.yarn.event.AsyncDispatcher;
|
|
|
import org.apache.hadoop.yarn.event.Dispatcher;
|
|
|
import org.apache.hadoop.yarn.event.EventHandler;
|
|
|
import org.apache.hadoop.yarn.exceptions.YarnException;
|
|
|
import org.apache.hadoop.yarn.exceptions.YarnRuntimeException;
|
|
|
import org.apache.hadoop.yarn.factories.RecordFactory;
|
|
|
import org.apache.hadoop.yarn.factory.providers.RecordFactoryProvider;
|
|
|
+import org.apache.hadoop.yarn.security.AMRMTokenIdentifier;
|
|
|
import org.apache.hadoop.yarn.server.api.ResourceTracker;
|
|
|
import org.apache.hadoop.yarn.server.api.protocolrecords.NodeHeartbeatRequest;
|
|
|
import org.apache.hadoop.yarn.server.api.protocolrecords.NodeHeartbeatResponse;
|
|
@@ -61,24 +63,31 @@ import org.apache.hadoop.yarn.server.api.records.NodeStatus;
|
|
|
import org.apache.hadoop.yarn.server.applicationhistoryservice.ApplicationHistoryServer;
|
|
|
import org.apache.hadoop.yarn.server.applicationhistoryservice.ApplicationHistoryStore;
|
|
|
import org.apache.hadoop.yarn.server.applicationhistoryservice.MemoryApplicationHistoryStore;
|
|
|
+import org.apache.hadoop.yarn.server.nodemanager.ContainerExecutor;
|
|
|
import org.apache.hadoop.yarn.server.nodemanager.Context;
|
|
|
+import org.apache.hadoop.yarn.server.nodemanager.DeletionService;
|
|
|
+import org.apache.hadoop.yarn.server.nodemanager.LocalDirsHandlerService;
|
|
|
import org.apache.hadoop.yarn.server.nodemanager.NodeHealthCheckerService;
|
|
|
import org.apache.hadoop.yarn.server.nodemanager.NodeManager;
|
|
|
import org.apache.hadoop.yarn.server.nodemanager.NodeStatusUpdater;
|
|
|
import org.apache.hadoop.yarn.server.nodemanager.NodeStatusUpdaterImpl;
|
|
|
+import org.apache.hadoop.yarn.server.nodemanager.amrmproxy.AMRMProxyService;
|
|
|
+import org.apache.hadoop.yarn.server.nodemanager.amrmproxy.DefaultRequestInterceptor;
|
|
|
+import org.apache.hadoop.yarn.server.nodemanager.amrmproxy.RequestInterceptor;
|
|
|
+import org.apache.hadoop.yarn.server.nodemanager.containermanager.ContainerManagerImpl;
|
|
|
+import org.apache.hadoop.yarn.server.nodemanager.metrics.NodeManagerMetrics;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceManager;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.ResourceTrackerService;
|
|
|
-import org.apache.hadoop.yarn.server.resourcemanager.reservation.ReservationSystemTestUtil;
|
|
|
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.event.RMAppAttemptRegistrationEvent;
|
|
|
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.event.RMAppAttemptUnregistrationEvent;
|
|
|
+import org.apache.hadoop.yarn.server.security.ApplicationACLsManager;
|
|
|
import org.apache.hadoop.yarn.server.timeline.MemoryTimelineStore;
|
|
|
import org.apache.hadoop.yarn.server.timeline.TimelineStore;
|
|
|
import org.apache.hadoop.yarn.server.timeline.recovery.MemoryTimelineStateStore;
|
|
|
import org.apache.hadoop.yarn.server.timeline.recovery.TimelineStateStore;
|
|
|
import org.apache.hadoop.yarn.util.timeline.TimelineUtils;
|
|
|
-import org.apache.hadoop.yarn.util.resource.Resources;
|
|
|
import org.apache.hadoop.yarn.webapp.util.WebAppUtils;
|
|
|
|
|
|
import com.google.common.annotations.VisibleForTesting;
|
|
@@ -698,6 +707,15 @@ public class MiniYARNCluster extends CompositeService {
|
|
|
protected void stopRMProxy() { }
|
|
|
};
|
|
|
}
|
|
|
+
|
|
|
+ @Override
|
|
|
+ protected ContainerManagerImpl createContainerManager(Context context,
|
|
|
+ ContainerExecutor exec, DeletionService del,
|
|
|
+ NodeStatusUpdater nodeStatusUpdater, ApplicationACLsManager aclsManager,
|
|
|
+ LocalDirsHandlerService dirsHandler) {
|
|
|
+ return new CustomContainerManagerImpl(context, exec, del,
|
|
|
+ nodeStatusUpdater, metrics, dirsHandler);
|
|
|
+ }
|
|
|
}
|
|
|
|
|
|
/**
|
|
@@ -799,4 +817,55 @@ public class MiniYARNCluster extends CompositeService {
|
|
|
public int getNumOfResourceManager() {
|
|
|
return this.resourceManagers.length;
|
|
|
}
|
|
|
+
|
|
|
+ private class CustomContainerManagerImpl extends ContainerManagerImpl {
|
|
|
+
|
|
|
+ public CustomContainerManagerImpl(Context context, ContainerExecutor exec,
|
|
|
+ DeletionService del, NodeStatusUpdater nodeStatusUpdater,
|
|
|
+ NodeManagerMetrics metrics, LocalDirsHandlerService dirsHandler) {
|
|
|
+ super(context, exec, del, nodeStatusUpdater, metrics, dirsHandler);
|
|
|
+ }
|
|
|
+
|
|
|
+ @Override
|
|
|
+ protected void createAMRMProxyService(Configuration conf) {
|
|
|
+ this.amrmProxyEnabled =
|
|
|
+ conf.getBoolean(YarnConfiguration.AMRM_PROXY_ENABLED,
|
|
|
+ YarnConfiguration.DEFAULT_AMRM_PROXY_ENABLED);
|
|
|
+
|
|
|
+ if (this.amrmProxyEnabled) {
|
|
|
+ LOG.info("CustomAMRMProxyService is enabled. "
|
|
|
+ + "All the AM->RM requests will be intercepted by the proxy");
|
|
|
+ AMRMProxyService amrmProxyService =
|
|
|
+ useRpc ? new AMRMProxyService(getContext(), dispatcher)
|
|
|
+ : new ShortCircuitedAMRMProxy(getContext(), dispatcher);
|
|
|
+ this.setAMRMProxyService(amrmProxyService);
|
|
|
+ addService(this.getAMRMProxyService());
|
|
|
+ } else {
|
|
|
+ LOG.info("CustomAMRMProxyService is disabled");
|
|
|
+ }
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ private class ShortCircuitedAMRMProxy extends AMRMProxyService {
|
|
|
+
|
|
|
+ public ShortCircuitedAMRMProxy(Context context,
|
|
|
+ AsyncDispatcher dispatcher) {
|
|
|
+ super(context, dispatcher);
|
|
|
+ }
|
|
|
+
|
|
|
+ @Override
|
|
|
+ protected void initializePipeline(ApplicationAttemptId applicationAttemptId,
|
|
|
+ String user, Token<AMRMTokenIdentifier> amrmToken,
|
|
|
+ Token<AMRMTokenIdentifier> localToken) {
|
|
|
+ super.initializePipeline(applicationAttemptId, user, amrmToken,
|
|
|
+ localToken);
|
|
|
+ RequestInterceptor rt = getPipelines()
|
|
|
+ .get(applicationAttemptId.getApplicationId()).getRootInterceptor();
|
|
|
+ if (rt instanceof DefaultRequestInterceptor) {
|
|
|
+ ((DefaultRequestInterceptor) rt)
|
|
|
+ .setRMClient(getResourceManager().getApplicationMasterService());
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ }
|
|
|
}
|