YARN-11465. Improved YarnClient Log Format (#5550)

Co-authored-by: yl09099 <shaq376260428@163.com>
Reviewed-by: Shilun Fan <slfan1989@apache.org>
Signed-off-by: Shilun Fan <slfan1989@apache.org>
This commit is contained in:
yl09099 2023-04-17 09:27:52 +08:00 committed by GitHub
parent 0d1b4a3556
commit 2c4d6bf33d
No known key found for this signature in database
GPG Key ID: 4AEE18F83AFDEB23
9 changed files with 82 additions and 89 deletions

View File

@ -69,17 +69,16 @@ public void onText(Session session, String message) throws IOException {
@OnWebSocketConnect @OnWebSocketConnect
public void onConnect(Session s) { public void onConnect(Session s) {
initTerminal(s); initTerminal(s);
LOG.info(s.getRemoteAddress().getHostString() + " connected!"); LOG.info("{} connected!", s.getRemoteAddress().getHostString());
} }
@OnWebSocketClose @OnWebSocketClose
public void onClose(Session session, int status, String reason) { public void onClose(Session session, int status, String reason) {
if (status==1000) { if (status==1000) {
LOG.info(session.getRemoteAddress().getHostString() + LOG.info("{} closed, status: {}", session.getRemoteAddress().getHostString(), status);
" closed, status: " + status);
} else { } else {
LOG.warn(session.getRemoteAddress().getHostString() + LOG.warn("{} closed, status:" +
" closed, status: " + status + " Reason: " + reason); " {} Reason: {}.", session.getRemoteAddress().getHostString(), status, reason);
} }
} }

View File

@ -133,7 +133,7 @@ protected void serviceInit(Configuration conf) throws Exception {
this.maxThreadPoolSize = conf.getInt( this.maxThreadPoolSize = conf.getInt(
YarnConfiguration.NM_CLIENT_ASYNC_THREAD_POOL_MAX_SIZE, YarnConfiguration.NM_CLIENT_ASYNC_THREAD_POOL_MAX_SIZE,
YarnConfiguration.DEFAULT_NM_CLIENT_ASYNC_THREAD_POOL_MAX_SIZE); YarnConfiguration.DEFAULT_NM_CLIENT_ASYNC_THREAD_POOL_MAX_SIZE);
LOG.info("Upper bound of the thread pool size is " + maxThreadPoolSize); LOG.info("Upper bound of the thread pool size is {}.", maxThreadPoolSize);
client.init(conf); client.init(conf);
super.serviceInit(conf); super.serviceInit(conf);
@ -186,9 +186,8 @@ public void run() {
// always increasing the pool-size // always increasing the pool-size
int newThreadPoolSize = Math.min(maxThreadPoolSize, int newThreadPoolSize = Math.min(maxThreadPoolSize,
idealThreadPoolSize + INITIAL_THREAD_POOL_SIZE); idealThreadPoolSize + INITIAL_THREAD_POOL_SIZE);
LOG.info("Set NMClientAsync thread pool size to " + LOG.info("Set NMClientAsync thread pool size to {} " +
newThreadPoolSize + " as the number of nodes to talk to is " "as the number of nodes to talk to is {}.", newThreadPoolSize, nodeNum);
+ nodeNum);
threadPool.setCorePoolSize(newThreadPoolSize); threadPool.setCorePoolSize(newThreadPoolSize);
} }
} }
@ -252,8 +251,7 @@ public void startContainerAsync(
try { try {
events.put(new StartContainerEvent(container, containerLaunchContext)); events.put(new StartContainerEvent(container, containerLaunchContext));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event of starting Container " + LOG.warn("Exception when scheduling the event of starting Container {}", container.getId());
container.getId());
callbackHandler.onStartContainerError(container.getId(), e); callbackHandler.onStartContainerError(container.getId(), e);
} }
} }
@ -276,8 +274,8 @@ public void increaseContainerResourceAsync(Container container) {
try { try {
events.put(new UpdateContainerResourceEvent(container, true)); events.put(new UpdateContainerResourceEvent(container, true));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event of increasing resource of " LOG.warn("Exception when scheduling the event of increasing " +
+ "Container " + container.getId()); "resource of Container {}", container.getId());
handler.onIncreaseContainerResourceError(container.getId(), e); handler.onIncreaseContainerResourceError(container.getId(), e);
} }
} }
@ -300,8 +298,8 @@ public void updateContainerResourceAsync(Container container) {
try { try {
events.put(new UpdateContainerResourceEvent(container, false)); events.put(new UpdateContainerResourceEvent(container, false));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event of increasing resource of " LOG.warn("Exception when scheduling the event of " +
+ "Container " + container.getId()); "increasing resource of Container {}.", container.getId());
handler.onUpdateContainerResourceError(container.getId(), e); handler.onUpdateContainerResourceError(container.getId(), e);
} }
} }
@ -325,8 +323,8 @@ public void reInitializeContainerAsync(ContainerId containerId,
client.getNodeIdOfStartedContainer(containerId), client.getNodeIdOfStartedContainer(containerId),
containerLaunchContex, autoCommit)); containerLaunchContex, autoCommit));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event of re-initializing of " LOG.warn("Exception when scheduling the event of " +
+ "Container " + containerId); "re-initializing of Container {}", containerId);
handler.onContainerReInitializeError(containerId, e); handler.onContainerReInitializeError(containerId, e);
} }
} }
@ -349,8 +347,7 @@ public void restartContainerAsync(ContainerId containerId){
client.getNodeIdOfStartedContainer(containerId), client.getNodeIdOfStartedContainer(containerId),
null, ContainerEventType.RESTART_CONTAINER)); null, ContainerEventType.RESTART_CONTAINER));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event of restart of " LOG.warn("Exception when scheduling the event of restart of Container {}", containerId);
+ "Container " + containerId);
handler.onContainerRestartError(containerId, e); handler.onContainerRestartError(containerId, e);
} }
} }
@ -373,8 +370,8 @@ public void rollbackLastReInitializationAsync(ContainerId containerId){
client.getNodeIdOfStartedContainer(containerId), client.getNodeIdOfStartedContainer(containerId),
null, ContainerEventType.ROLLBACK_LAST_REINIT)); null, ContainerEventType.ROLLBACK_LAST_REINIT));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event Rollback re-initialization" LOG.warn("Exception when scheduling the event Rollback " +
+ " of Container " + containerId); "re-initialization of Container {}", containerId);
handler.onRollbackLastReInitializationError(containerId, e); handler.onRollbackLastReInitializationError(containerId, e);
} }
} }
@ -397,8 +394,8 @@ public void commitLastReInitializationAsync(ContainerId containerId){
client.getNodeIdOfStartedContainer(containerId), client.getNodeIdOfStartedContainer(containerId),
null, ContainerEventType.COMMIT_LAST_REINT)); null, ContainerEventType.COMMIT_LAST_REINT));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event Commit re-initialization" LOG.warn("Exception when scheduling the event " +
+ " of Container " + containerId); "Commit re-initialization of Container {}", containerId);
handler.onCommitLastReInitializationError(containerId, e); handler.onCommitLastReInitializationError(containerId, e);
} }
} }
@ -413,8 +410,7 @@ public void stopContainerAsync(ContainerId containerId, NodeId nodeId) {
events.put(new ContainerEvent(containerId, nodeId, null, events.put(new ContainerEvent(containerId, nodeId, null,
ContainerEventType.STOP_CONTAINER)); ContainerEventType.STOP_CONTAINER));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event of stopping Container " + LOG.warn("Exception when scheduling the event of stopping Container {}", containerId);
containerId);
callbackHandler.onStopContainerError(containerId, e); callbackHandler.onStopContainerError(containerId, e);
} }
} }
@ -424,8 +420,8 @@ public void getContainerStatusAsync(ContainerId containerId, NodeId nodeId) {
events.put(new ContainerEvent(containerId, nodeId, null, events.put(new ContainerEvent(containerId, nodeId, null,
ContainerEventType.QUERY_CONTAINER)); ContainerEventType.QUERY_CONTAINER));
} catch (InterruptedException e) { } catch (InterruptedException e) {
LOG.warn("Exception when scheduling the event of querying the status" + LOG.warn("Exception when scheduling the event of querying " +
" of Container " + containerId); "the status of Container {}", containerId);
callbackHandler.onGetContainerStatusError(containerId, e); callbackHandler.onGetContainerStatusError(containerId, e);
} }
} }
@ -730,7 +726,7 @@ public ContainerState transition(StatefulContainer container,
switch(containerEvent.getType()) { switch(containerEvent.getType()) {
case REINITIALIZE_CONTAINER: case REINITIALIZE_CONTAINER:
if (!(containerEvent instanceof ReInitializeContainerEvevnt)) { if (!(containerEvent instanceof ReInitializeContainerEvevnt)) {
LOG.error("Unexpected Event.. [" +containerEvent.getType() + "]"); LOG.error("Unexpected Event.. [{}]", containerEvent.getType());
return ContainerState.FAILED; return ContainerState.FAILED;
} }
ReInitializeContainerEvevnt rEvent = ReInitializeContainerEvevnt rEvent =
@ -771,8 +767,8 @@ public ContainerState transition(StatefulContainer container,
} }
break; break;
default: default:
LOG.warn("Event of type [" + containerEvent.getType() + "] not" + LOG.warn("Event of type [{}] not" +
" expected here.."); " expected here..", containerEvent.getType());
break; break;
} }
if (handlerError != null) { if (handlerError != null) {
@ -942,7 +938,7 @@ public ContainerEventProcessor(ContainerEvent event) {
@Override @Override
public void run() { public void run() {
ContainerId containerId = event.getContainerId(); ContainerId containerId = event.getContainerId();
LOG.info("Processing Event " + event + " for Container " + containerId); LOG.info("Processing Event {} for Container {}", event, containerId);
if (event.getType() == ContainerEventType.QUERY_CONTAINER) { if (event.getType() == ContainerEventType.QUERY_CONTAINER) {
try { try {
ContainerStatus containerStatus = client.getContainerStatus( ContainerStatus containerStatus = client.getContainerStatus(
@ -962,7 +958,7 @@ public void run() {
} else { } else {
StatefulContainer container = containers.get(containerId); StatefulContainer container = containers.get(containerId);
if (container == null) { if (container == null) {
LOG.info("Container " + containerId + " is already stopped or failed"); LOG.info("Container {} is already stopped or failed", containerId);
} else { } else {
container.handle(event); container.handle(event);
if (isCompletelyDone(container)) { if (isCompletelyDone(container)) {

View File

@ -478,11 +478,12 @@ protected void removePendingChangeRequests(
continue; continue;
} }
if (LOG.isDebugEnabled()) { if (LOG.isDebugEnabled()) {
LOG.debug("RM has confirmed changed resource allocation for " LOG.debug("RM has confirmed changed resource allocation for container {}. " +
+ "container " + containerId + ". Current resource allocation:" "Current resource allocation:{}. " +
+ changedContainer.getContainer().getResource() "Remove pending change request:{}",
+ ". Remove pending change request:" containerId,
+ pendingChange.get(containerId).getValue()); changedContainer.getContainer().getResource(),
pendingChange.get(containerId).getValue());
} }
pendingChange.remove(containerId); pendingChange.remove(containerId);
} }
@ -495,9 +496,9 @@ protected void populateNMTokens(List<NMToken> nmTokens) {
String nodeId = token.getNodeId().toString(); String nodeId = token.getNodeId().toString();
if (LOG.isDebugEnabled()) { if (LOG.isDebugEnabled()) {
if (getNMTokenCache().containsToken(nodeId)) { if (getNMTokenCache().containsToken(nodeId)) {
LOG.debug("Replacing token for : " + nodeId); LOG.debug("Replacing token for : {}", nodeId);
} else { } else {
LOG.debug("Received new token for : " + nodeId); LOG.debug("Received new token for : {}", nodeId);
} }
} }
getNMTokenCache().setToken(nodeId, token.getToken()); getNMTokenCache().setToken(nodeId, token.getToken());
@ -544,8 +545,7 @@ public synchronized void addContainerRequest(T req) {
dedupedRacks.addAll(req.getRacks()); dedupedRacks.addAll(req.getRacks());
if(req.getRacks().size() != dedupedRacks.size()) { if(req.getRacks().size() != dedupedRacks.size()) {
Joiner joiner = Joiner.on(','); Joiner joiner = Joiner.on(',');
LOG.warn("ContainerRequest has duplicate racks: " LOG.warn("ContainerRequest has duplicate racks: {}", joiner.join(req.getRacks()));
+ joiner.join(req.getRacks()));
} }
} }
Set<String> inferredRacks = resolveRacks(req.getNodes()); Set<String> inferredRacks = resolveRacks(req.getNodes());
@ -573,8 +573,7 @@ public synchronized void addContainerRequest(T req) {
HashSet<String> dedupedNodes = new HashSet<String>(req.getNodes()); HashSet<String> dedupedNodes = new HashSet<String>(req.getNodes());
if(dedupedNodes.size() != req.getNodes().size()) { if(dedupedNodes.size() != req.getNodes().size()) {
Joiner joiner = Joiner.on(','); Joiner joiner = Joiner.on(',');
LOG.warn("ContainerRequest has duplicate nodes: " LOG.warn("ContainerRequest has duplicate nodes: {}", joiner.join(req.getNodes()));
+ joiner.join(req.getNodes()));
} }
for (String node : dedupedNodes) { for (String node : dedupedNodes) {
addResourceRequest(req.getPriority(), node, addResourceRequest(req.getPriority(), node,
@ -636,11 +635,12 @@ public synchronized void requestContainerUpdate(
Preconditions.checkNotNull(container, "Container cannot be null!!"); Preconditions.checkNotNull(container, "Container cannot be null!!");
Preconditions.checkNotNull(updateContainerRequest, Preconditions.checkNotNull(updateContainerRequest,
"UpdateContainerRequest cannot be null!!"); "UpdateContainerRequest cannot be null!!");
LOG.info("Requesting Container update : " + LOG.info("Requesting Container update : container={}, updateType={}," +
"container=" + container + ", " + " targetCapability={}, targetExecType={}",
"updateType=" + updateContainerRequest.getContainerUpdateType() + ", " + container,
"targetCapability=" + updateContainerRequest.getCapability() + ", " + updateContainerRequest.getContainerUpdateType(),
"targetExecType=" + updateContainerRequest.getExecutionType()); updateContainerRequest.getCapability(),
updateContainerRequest.getExecutionType());
if (updateContainerRequest.getCapability() != null && if (updateContainerRequest.getCapability() != null &&
updateContainerRequest.getExecutionType() == null) { updateContainerRequest.getExecutionType() == null) {
validateContainerResourceChangeRequest( validateContainerResourceChangeRequest(
@ -770,7 +770,7 @@ private Set<String> resolveRacks(List<String> nodes) {
// Ensure node requests are accompanied by requests for // Ensure node requests are accompanied by requests for
// corresponding rack // corresponding rack
if (rack == null) { if (rack == null) {
LOG.warn("Failed to resolve rack for node " + node + "."); LOG.warn("Failed to resolve rack for node {}.", node);
} else { } else {
racks.add(rack); racks.add(rack);
} }
@ -941,12 +941,13 @@ private void addResourceRequest(Priority priority, String resourceName,
addResourceRequestToAsk(resourceRequestInfo.remoteRequest); addResourceRequestToAsk(resourceRequestInfo.remoteRequest);
if (LOG.isDebugEnabled()) { if (LOG.isDebugEnabled()) {
LOG.debug("Adding request to ask " + resourceRequestInfo.remoteRequest); LOG.debug("Adding request to ask {}", resourceRequestInfo.remoteRequest);
LOG.debug("addResourceRequest:" + " applicationId=" LOG.debug("addResourceRequest: applicationId= priority={}" +
+ " priority=" + priority.getPriority() " resourceName={} numContainers={} #asks={}",
+ " resourceName=" + resourceName + " numContainers=" priority.getPriority(),
+ resourceRequestInfo.remoteRequest.getNumContainers() resourceName,
+ " #asks=" + ask.size()); resourceRequestInfo.remoteRequest.getNumContainers(),
ask.size());
} }
} }
@ -972,17 +973,16 @@ private void decResourceRequest(Priority priority, String resourceName,
} }
if (LOG.isDebugEnabled()) { if (LOG.isDebugEnabled()) {
LOG.debug("AFTER decResourceRequest:" LOG.debug("AFTER decResourceRequest: allocationRequestId={} " +
+ " allocationRequestId=" + req.getAllocationRequestId() "priority={} resourceName={} numContainers={} #asks={}",
+ " priority=" + priority.getPriority() req.getAllocationRequestId(), priority.getPriority(),
+ " resourceName=" + resourceName + " numContainers=" resourceName,
+ resourceRequestInfo.remoteRequest.getNumContainers() resourceRequestInfo.remoteRequest.getNumContainers(), ask.size());
+ " #asks=" + ask.size());
} }
} }
} else { } else {
LOG.info("No remoteRequestTable found with allocationRequestId=" LOG.info("No remoteRequestTable found with allocationRequestId={}",
+ req.getAllocationRequestId()); req.getAllocationRequestId());
} }
} }

View File

@ -128,13 +128,11 @@ protected synchronized void cleanupRunningContainers() {
stopContainer(startedContainer.getContainerId(), stopContainer(startedContainer.getContainerId(),
startedContainer.getNodeId()); startedContainer.getNodeId());
} catch (YarnException e) { } catch (YarnException e) {
LOG.error("Failed to stop Container " + LOG.error("Failed to stop Container {} when stopping NMClientImpl",
startedContainer.getContainerId() + startedContainer.getContainerId());
" when stopping NMClientImpl");
} catch (IOException e) { } catch (IOException e) {
LOG.error("Failed to stop Container " + LOG.error("Failed to stop Container {} when stopping NMClientImpl",
startedContainer.getContainerId() + startedContainer.getContainerId());
" when stopping NMClientImpl");
} }
} }
} }

View File

@ -353,7 +353,7 @@ public YarnClientApplication createApplication()
throw new YarnException("Failed to submit " + applicationId + throw new YarnException("Failed to submit " + applicationId +
" to YARN : " + appReport.getDiagnostics()); " to YARN : " + appReport.getDiagnostics());
} }
LOG.info("Submitted application " + applicationId); LOG.info("Submitted application {}", applicationId);
break; break;
} }
@ -368,8 +368,9 @@ public YarnClientApplication createApplication()
// is blocked here too long. // is blocked here too long.
if (++pollCount % 10 == 0) { if (++pollCount % 10 == 0) {
LOG.info("Application submission is not finished, " + LOG.info("Application submission is not finished, " +
"submitted application " + applicationId + "submitted application {} is still in {}",
" is still in " + state); applicationId,
state);
} }
try { try {
Thread.sleep(submitPollIntervalMillis); Thread.sleep(submitPollIntervalMillis);
@ -382,8 +383,8 @@ public YarnClientApplication createApplication()
} catch (ApplicationNotFoundException ex) { } catch (ApplicationNotFoundException ex) {
// FailOver or RM restart happens before RMStateStore saves // FailOver or RM restart happens before RMStateStore saves
// ApplicationState // ApplicationState
LOG.info("Re-submit application " + applicationId + "with the " + LOG.info("Re-submit application {} with the" +
"same ApplicationSubmissionContext"); " same ApplicationSubmissionContext", applicationId);
rmClient.submitApplication(request); rmClient.submitApplication(request);
} }
} }
@ -408,7 +409,7 @@ private void addLogAggregationDelegationToken(
throw new IOException( throw new IOException(
"Can't get Master Kerberos principal for use as renewer"); "Can't get Master Kerberos principal for use as renewer");
} }
LOG.debug("Delegation Token Renewer: " + masterPrincipal); LOG.debug("Delegation Token Renewer: {}", masterPrincipal);
LogAggregationFileControllerFactory factory = LogAggregationFileControllerFactory factory =
new LogAggregationFileControllerFactory(conf); new LogAggregationFileControllerFactory(conf);
@ -421,8 +422,7 @@ private void addLogAggregationDelegationToken(
fs.addDelegationTokens(masterPrincipal, credentials); fs.addDelegationTokens(masterPrincipal, credentials);
if (finalTokens != null) { if (finalTokens != null) {
for (org.apache.hadoop.security.token.Token<?> token : finalTokens) { for (org.apache.hadoop.security.token.Token<?> token : finalTokens) {
LOG.info("Added delegation token for log aggregation path " LOG.info("Added delegation token for log aggregation path {}; {}", remoteRootLogDir, token);
+ remoteRootLogDir + "; "+token);
} }
} }
@ -485,8 +485,7 @@ private void addTimelineDelegationToken(
return timelineClient.getDelegationToken(timelineDTRenewer); return timelineClient.getDelegationToken(timelineDTRenewer);
} catch (Exception e) { } catch (Exception e) {
if (timelineServiceBestEffort) { if (timelineServiceBestEffort) {
LOG.warn("Failed to get delegation token from the timeline server: " LOG.warn("Failed to get delegation token from the timeline server: {}", e.getMessage());
+ e.getMessage());
return null; return null;
} }
throw new IOException(e); throw new IOException(e);
@ -527,7 +526,7 @@ protected boolean isSecurityEnabled() {
@Override @Override
public void failApplicationAttempt(ApplicationAttemptId attemptId) public void failApplicationAttempt(ApplicationAttemptId attemptId)
throws YarnException, IOException { throws YarnException, IOException {
LOG.info("Failing application attempt " + attemptId); LOG.info("Failing application attempt {}.", attemptId);
FailApplicationAttemptRequest request = FailApplicationAttemptRequest request =
Records.newRecord(FailApplicationAttemptRequest.class); Records.newRecord(FailApplicationAttemptRequest.class);
request.setApplicationAttemptId(attemptId); request.setApplicationAttemptId(attemptId);
@ -560,7 +559,7 @@ public void killApplication(ApplicationId applicationId, String diagnostics)
KillApplicationResponse response = KillApplicationResponse response =
rmClient.forceKillApplication(request); rmClient.forceKillApplication(request);
if (response.getIsKillCompleted()) { if (response.getIsKillCompleted()) {
LOG.info("Killed application " + applicationId); LOG.info("Killed application {}", applicationId);
break; break;
} }
@ -573,7 +572,7 @@ public void killApplication(ApplicationId applicationId, String diagnostics)
if (++pollCount % 10 == 0) { if (++pollCount % 10 == 0) {
LOG.info( LOG.info(
"Waiting for application " + applicationId + " to be killed."); "Waiting for application {} to be killed.", applicationId);
} }
Thread.sleep(asyncApiPollIntervalMillis); Thread.sleep(asyncApiPollIntervalMillis);
} }
@ -1080,7 +1079,7 @@ public Priority updateApplicationPriority(ApplicationId applicationId,
public void signalToContainer(ContainerId containerId, public void signalToContainer(ContainerId containerId,
SignalContainerCommand command) SignalContainerCommand command)
throws YarnException, IOException { throws YarnException, IOException {
LOG.info("Signalling container " + containerId + " with command " + command); LOG.info("Signalling container {} with command {}", containerId, command);
SignalContainerRequest request = SignalContainerRequest request =
SignalContainerRequest.newInstance(containerId, command); SignalContainerRequest.newInstance(containerId, command);
rmClient.signalToContainer(request); rmClient.signalToContainer(request);
@ -1186,9 +1185,9 @@ public void shellToContainer(ContainerId containerId,
client.stop(); client.stop();
} }
} catch (WebSocketException e) { } catch (WebSocketException e) {
LOG.debug("Websocket exception: " + e.getMessage()); LOG.debug("Websocket exception: {}", e.getMessage());
} catch (Throwable t) { } catch (Throwable t) {
LOG.error("Fail to shell to container: " + t.getMessage()); LOG.error("Fail to shell to container: {}", t.getMessage());
} }
} }
} }

View File

@ -79,7 +79,7 @@ public void run() {
boolean rmStarted = rmStartedSignal.await(60000L, TimeUnit.MILLISECONDS); boolean rmStarted = rmStartedSignal.await(60000L, TimeUnit.MILLISECONDS);
Assert.assertTrue("ResourceManager failed to start up.", rmStarted); Assert.assertTrue("ResourceManager failed to start up.", rmStarted);
LOG.info("ResourceManager RMAdmin address: " + LOG.info("ResourceManager RMAdmin address: {}.",
conf.get(YarnConfiguration.RM_ADMIN_ADDRESS)); conf.get(YarnConfiguration.RM_ADMIN_ADDRESS));
} }

View File

@ -189,6 +189,7 @@ public void testFailoverChange() throws Exception {
* and {@link AutoRefreshRMFailoverProxyProvider#performFailover(Object)} * and {@link AutoRefreshRMFailoverProxyProvider#performFailover(Object)}
* gets called. * gets called.
*/ */
@SuppressWarnings("unchecked")
@Test @Test
public void testAutoRefreshFailoverChange() throws Exception { public void testAutoRefreshFailoverChange() throws Exception {
conf.setClass(YarnConfiguration.CLIENT_FAILOVER_NO_HA_PROXY_PROVIDER, conf.setClass(YarnConfiguration.CLIENT_FAILOVER_NO_HA_PROXY_PROVIDER,

View File

@ -106,8 +106,8 @@ public void run() {
boolean rmStarted = rmStartedSignal.await(60000L, TimeUnit.MILLISECONDS); boolean rmStarted = rmStartedSignal.await(60000L, TimeUnit.MILLISECONDS);
Assert.assertTrue("ResourceManager failed to start up.", rmStarted); Assert.assertTrue("ResourceManager failed to start up.", rmStarted);
LOG.info("ResourceManager RMAdmin address: " LOG.info("ResourceManager RMAdmin address: {}.",
+ configuration.get(YarnConfiguration.RM_ADMIN_ADDRESS)); configuration.get(YarnConfiguration.RM_ADMIN_ADDRESS));
client = new ResourceManagerAdministrationProtocolPBClientImpl(1L, client = new ResourceManagerAdministrationProtocolPBClientImpl(1L,
getProtocolAddress(configuration), configuration); getProtocolAddress(configuration), configuration);

View File

@ -76,7 +76,7 @@ public static void afterClass() {
localFs.close(); localFs.close();
} }
} catch (IOException ioe) { } catch (IOException ioe) {
LOG.info("IO exception in closing file system)"); LOG.info("IO exception in closing file system");
ioe.printStackTrace(); ioe.printStackTrace();
} }
} }