Skip to content

Commit

Permalink
YARN-9317. Avoid repeated YarnConfiguration#timelineServiceV2Enabled …
Browse files Browse the repository at this point in the history
…check. Contributed by Prabhu Joseph
  • Loading branch information
bibinchundatt committed Feb 22, 2019
1 parent 95fbbfe commit ed13cf8
Show file tree
Hide file tree
Showing 8 changed files with 41 additions and 17 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -160,6 +160,7 @@ public class NodeStatusUpdaterImpl extends AbstractService implements
private NodeLabelsProvider nodeLabelsProvider;
private NodeAttributesProvider nodeAttributesProvider;
private long tokenSequenceNo;
private boolean timelineServiceV2Enabled;

public NodeStatusUpdaterImpl(Context context, Dispatcher dispatcher,
NodeHealthCheckerService healthChecker, NodeManagerMetrics metrics) {
Expand Down Expand Up @@ -254,6 +255,9 @@ protected void serviceInit(Configuration conf) throws Exception {
this.logAggregationEnabled =
conf.getBoolean(YarnConfiguration.LOG_AGGREGATION_ENABLED,
YarnConfiguration.DEFAULT_LOG_AGGREGATION_ENABLED);
this.timelineServiceV2Enabled = YarnConfiguration.
timelineServiceV2Enabled(conf);

}

@Override
Expand Down Expand Up @@ -1411,7 +1415,7 @@ public void run() {
newResource.toString());
}
}
if (YarnConfiguration.timelineServiceV2Enabled(context.getConf())) {
if (timelineServiceV2Enabled) {
updateTimelineCollectorData(response);
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -230,6 +230,7 @@ private enum ReInitOp {

// NM metrics publisher is set only if the timeline service v.2 is enabled
private NMTimelinePublisher nmMetricsPublisher;
private boolean timelineServiceV2Enabled;

public ContainerManagerImpl(Context context, ContainerExecutor exec,
DeletionService deletionContext, NodeStatusUpdater nodeStatusUpdater,
Expand Down Expand Up @@ -267,11 +268,13 @@ public ContainerManagerImpl(Context context, ContainerExecutor exec,
// initialize the metrics publisher if the timeline service v.2 is enabled
// and the system publisher is enabled
Configuration conf = context.getConf();
if (YarnConfiguration.timelineServiceV2Enabled(conf) &&
YarnConfiguration.systemMetricsPublisherEnabled(conf)) {
LOG.info("YARN system metrics publishing service is enabled");
nmMetricsPublisher = createNMTimelinePublisher(context);
context.setNMTimelinePublisher(nmMetricsPublisher);
if (YarnConfiguration.timelineServiceV2Enabled(conf)) {
if (YarnConfiguration.systemMetricsPublisherEnabled(conf)) {
LOG.info("YARN system metrics publishing service is enabled");
nmMetricsPublisher = createNMTimelinePublisher(context);
context.setNMTimelinePublisher(nmMetricsPublisher);
}
this.timelineServiceV2Enabled = true;
}
this.containersMonitor = createContainersMonitor(exec);
addService(this.containersMonitor);
Expand Down Expand Up @@ -1191,7 +1194,7 @@ protected void startContainerInternal(
private FlowContext getFlowContext(ContainerLaunchContext launchContext,
ApplicationId applicationID) {
FlowContext flowContext = null;
if (YarnConfiguration.timelineServiceV2Enabled(getConfig())) {
if (timelineServiceV2Enabled) {
String flowName = launchContext.getEnvironment()
.get(TimelineUtils.FLOW_NAME_TAG_PREFIX);
String flowVersion = launchContext.getEnvironment()
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -95,6 +95,7 @@ public class ApplicationMasterService extends AbstractService implements
new ConcurrentHashMap<ApplicationAttemptId, AllocateResponseLock>();
protected final RMContext rmContext;
private final AMSProcessingChain amsProcessingChain;
private boolean timelineServiceV2Enabled;

public ApplicationMasterService(RMContext rmContext,
YarnScheduler scheduler) {
Expand Down Expand Up @@ -212,6 +213,9 @@ protected void serviceStart() throws Exception {
YarnConfiguration.RM_SCHEDULER_ADDRESS,
YarnConfiguration.DEFAULT_RM_SCHEDULER_ADDRESS,
server.getListenerAddress());
this.timelineServiceV2Enabled = YarnConfiguration.
timelineServiceV2Enabled(conf);

super.serviceStart();
}

Expand Down Expand Up @@ -302,7 +306,7 @@ public FinishApplicationMasterResponse finishApplicationMaster(
rmContext.getRMApps().get(applicationAttemptId.getApplicationId());

// Remove collector address when app get finished.
if (YarnConfiguration.timelineServiceV2Enabled(getConfig())) {
if (timelineServiceV2Enabled) {
((RMAppImpl) rmApp).removeCollectorData();
}
// checking whether the app exits in RMStateStore at first not to throw
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -236,6 +236,7 @@ public class ClientRMService extends AbstractService implements
RMAppState.ACCEPTED, RMAppState.RUNNING);

private ResourceProfilesManager resourceProfilesManager;
private boolean timelineServiceV2Enabled;

public ClientRMService(RMContext rmContext, YarnScheduler scheduler,
RMAppManager rmAppManager, ApplicationACLsManager applicationACLsManager,
Expand Down Expand Up @@ -306,6 +307,9 @@ protected void serviceStart() throws Exception {
YarnConfiguration.RM_ADDRESS,
YarnConfiguration.DEFAULT_RM_ADDRESS,
server.getListenerAddress());
this.timelineServiceV2Enabled = YarnConfiguration.
timelineServiceV2Enabled(conf);

super.serviceStart();
}

Expand Down Expand Up @@ -585,7 +589,7 @@ public SubmitApplicationResponse submitApplication(
throw RPCUtil.getRemoteException(ie);
}

if (YarnConfiguration.timelineServiceV2Enabled(getConfig())) {
if (timelineServiceV2Enabled) {
// Sanity check for flow run
String value = null;
try {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -115,12 +115,15 @@ final class DefaultAMSProcessor implements ApplicationMasterServiceProcessor {

private RMContext rmContext;
private ResourceProfilesManager resourceProfilesManager;
private boolean timelineServiceV2Enabled;

@Override
public void init(ApplicationMasterServiceContext amsContext,
ApplicationMasterServiceProcessor nextProcessor) {
this.rmContext = (RMContext)amsContext;
this.resourceProfilesManager = rmContext.getResourceProfilesManager();
this.timelineServiceV2Enabled = YarnConfiguration.
timelineServiceV2Enabled(rmContext.getYarnConfiguration());
}

@Override
Expand Down Expand Up @@ -326,8 +329,7 @@ public void allocate(ApplicationAttemptId appAttemptId,
response.setNumClusterNodes(getScheduler().getNumClusterNodes());

// add collector address for this application
if (YarnConfiguration.timelineServiceV2Enabled(
getRmContext().getYarnConfiguration())) {
if (timelineServiceV2Enabled) {
CollectorInfo collectorInfo = app.getCollectorInfo();
if (collectorInfo != null) {
response.setCollectorInfo(collectorInfo);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -95,6 +95,7 @@ public class RMAppManager implements EventHandler<RMAppManagerEvent>,
private final ApplicationACLsManager applicationACLsManager;
private Configuration conf;
private YarnAuthorizationProvider authorizer;
private boolean timelineServiceV2Enabled;

public RMAppManager(RMContext context,
YarnScheduler scheduler, ApplicationMasterService masterService,
Expand All @@ -115,6 +116,8 @@ public RMAppManager(RMContext context,
this.maxCompletedAppsInStateStore = this.maxCompletedAppsInMemory;
}
this.authorizer = YarnAuthorizationProvider.getInstance(conf);
this.timelineServiceV2Enabled = YarnConfiguration.
timelineServiceV2Enabled(conf);
}

/**
Expand Down Expand Up @@ -493,7 +496,7 @@ private RMAppImpl createAndPopulateNewRMApp(
throw new YarnException(message);
}

if (YarnConfiguration.timelineServiceV2Enabled(conf)) {
if (timelineServiceV2Enabled) {
// Start timeline collector for the submitted app
application.startTimelineCollector();
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -128,6 +128,7 @@ public class ResourceTrackerService extends AbstractService implements

private final AtomicLong timelineCollectorVersion = new AtomicLong(0);
private boolean checkIpHostnameInRegistration;
private boolean timelineServiceV2Enabled;

public ResourceTrackerService(RMContext rmContext,
NodesListManager nodesListManager,
Expand Down Expand Up @@ -177,6 +178,8 @@ protected void serviceInit(Configuration conf) throws Exception {
minimumNodeManagerVersion = conf.get(
YarnConfiguration.RM_NODEMANAGER_MINIMUM_VERSION,
YarnConfiguration.DEFAULT_RM_NODEMANAGER_MINIMUM_VERSION);
timelineServiceV2Enabled = YarnConfiguration.
timelineServiceV2Enabled(conf);

if (YarnConfiguration.areNodeLabelsEnabled(conf)) {
isDistributedNodeLabelsConf =
Expand Down Expand Up @@ -621,9 +624,7 @@ public NodeHeartbeatResponse nodeHeartbeat(NodeHeartbeatRequest request)
NodeAction.SHUTDOWN, message);
}

boolean timelineV2Enabled =
YarnConfiguration.timelineServiceV2Enabled(getConfig());
if (timelineV2Enabled) {
if (timelineServiceV2Enabled) {
// Check & update collectors info from request.
updateAppCollectorsMap(request);
}
Expand All @@ -639,7 +640,7 @@ public NodeHeartbeatResponse nodeHeartbeat(NodeHeartbeatRequest request)

populateTokenSequenceNo(request, nodeHeartBeatResponse);

if (timelineV2Enabled) {
if (timelineServiceV2Enabled) {
// Return collectors' map that NM needs to know
setAppCollectorsMapToResponse(rmNode.getRunningApps(),
nodeHeartBeatResponse);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -84,6 +84,7 @@ public class AMLauncher implements Runnable {
private final AMLauncherEventType eventType;
private final RMContext rmContext;
private final Container masterContainer;
private boolean timelineServiceV2Enabled;

@SuppressWarnings("rawtypes")
private final EventHandler handler;
Expand All @@ -96,6 +97,8 @@ public AMLauncher(RMContext rmContext, RMAppAttempt application,
this.rmContext = rmContext;
this.handler = rmContext.getDispatcher().getEventHandler();
this.masterContainer = application.getMasterContainer();
this.timelineServiceV2Enabled = YarnConfiguration.
timelineServiceV2Enabled(conf);
}

private void connect() throws IOException {
Expand Down Expand Up @@ -268,7 +271,7 @@ protected void setupTokens(
}

private void setFlowContext(ContainerLaunchContext container) {
if (YarnConfiguration.timelineServiceV2Enabled(conf)) {
if (timelineServiceV2Enabled) {
Map<String, String> environment = container.getEnvironment();
ApplicationId applicationId =
application.getAppAttemptId().getApplicationId();
Expand Down

0 comments on commit ed13cf8

Please sign in to comment.