YARN-8644. Improve unit test for RMAppImpl.FinalTransition. (Contributed by Szilard Nemeth)

This commit is contained in:
Haibo Chen 2018-10-05 09:31:48 -07:00
parent 73c660b43f
commit c968365650
2 changed files with 150 additions and 80 deletions

View File

@ -903,7 +903,6 @@ public class RMAppImpl implements RMApp, Recoverable {
@Override @Override
public void handle(RMAppEvent event) { public void handle(RMAppEvent event) {
this.writeLock.lock(); this.writeLock.lock();
try { try {
@ -1459,8 +1458,7 @@ public class RMAppImpl implements RMApp, Recoverable {
} }
} }
private static final class AppRejectedTransition extends private static final class AppRejectedTransition extends FinalTransition {
FinalTransition{
public AppRejectedTransition() { public AppRejectedTransition() {
super(RMAppState.FAILED); super(RMAppState.FAILED);
} }
@ -1502,39 +1500,50 @@ public class RMAppImpl implements RMApp, Recoverable {
private final RMAppState finalState; private final RMAppState finalState;
public FinalTransition(RMAppState finalState) { FinalTransition(RMAppState finalState) {
this.finalState = finalState; this.finalState = finalState;
} }
@Override
public void transition(RMAppImpl app, RMAppEvent event) { public void transition(RMAppImpl app, RMAppEvent event) {
app.logAggregationStartTime = app.systemClock.getTime(); completeAndCleanupApp(app);
handleAppFinished(app);
app.clearUnusedFields();
appAdminClientCleanUp(app);
}
private void completeAndCleanupApp(RMAppImpl app) {
//cleanup app in RM Nodes
for (NodeId nodeId : app.getRanNodes()) { for (NodeId nodeId : app.getRanNodes()) {
app.handler.handle( app.handler.handle(
new RMNodeCleanAppEvent(nodeId, app.applicationId)); new RMNodeCleanAppEvent(nodeId, app.applicationId));
} }
app.finishTime = app.storedFinishTime;
if (app.finishTime == 0 ) {
app.finishTime = app.systemClock.getTime();
}
// Recovered apps that are completed were not added to scheduler, so no // Recovered apps that are completed were not added to scheduler, so no
// need to remove them from scheduler. // need to remove them from scheduler.
if (app.recoveredFinalState == null) { if (app.recoveredFinalState == null) {
app.handler.handle(new AppRemovedSchedulerEvent(app.applicationId, app.handler.handle(new AppRemovedSchedulerEvent(app.applicationId,
finalState)); finalState));
} }
app.handler.handle(
new RMAppManagerEvent(app.applicationId,
RMAppManagerEventType.APP_COMPLETED));
// Send app completed event to AppManager
app.handler.handle(new RMAppManagerEvent(app.applicationId,
RMAppManagerEventType.APP_COMPLETED));
}
private void handleAppFinished(RMAppImpl app) {
app.logAggregationStartTime = app.systemClock.getTime();
// record finish time
app.finishTime = app.storedFinishTime;
if (app.finishTime == 0) {
app.finishTime = app.systemClock.getTime();
}
//record finish in history and metrics
app.rmContext.getRMApplicationHistoryWriter() app.rmContext.getRMApplicationHistoryWriter()
.applicationFinished(app, finalState); .applicationFinished(app, finalState);
app.rmContext.getSystemMetricsPublisher() app.rmContext.getSystemMetricsPublisher()
.appFinished(app, finalState, app.finishTime); .appFinished(app, finalState, app.finishTime);
// set the memory free }
app.clearUnusedFields();
appAdminClientCleanUp(app);
};
} }
public int getNumFailedAppAttempts() { public int getNumFailedAppAttempts() {
@ -1812,8 +1821,8 @@ public class RMAppImpl implements RMApp, Recoverable {
== LogAggregationStatus.TIME_OUT == LogAggregationStatus.TIME_OUT
&& report.getLogAggregationStatus() && report.getLogAggregationStatus()
== LogAggregationStatus.RUNNING) { == LogAggregationStatus.RUNNING) {
// If the log aggregation status got from latest nm heartbeat // If the log aggregation status got from latest NM heartbeat
// is Running, and current log aggregation status is TimeOut, // is RUNNING, and current log aggregation status is TIME_OUT,
// based on whether there are any failure messages for this NM, // based on whether there are any failure messages for this NM,
// we will reset the log aggregation status as RUNNING or // we will reset the log aggregation status as RUNNING or
// RUNNING_WITH_FAILURE // RUNNING_WITH_FAILURE
@ -2137,4 +2146,10 @@ public class RMAppImpl implements RMApp, Recoverable {
RMAppState state){ RMAppState state){
/* TODO fail the application on the failed transition */ /* TODO fail the application on the failed transition */
} }
@VisibleForTesting
public long getLogAggregationStartTime() {
return logAggregationStartTime;
}
} }

View File

@ -18,28 +18,7 @@
package org.apache.hadoop.yarn.server.resourcemanager.rmapp; package org.apache.hadoop.yarn.server.resourcemanager.rmapp;
import static org.junit.Assert.assertEquals; import com.google.common.collect.Lists;
import static org.mockito.Matchers.any;
import static org.mockito.Matchers.anyLong;
import static org.mockito.Matchers.eq;
import static org.mockito.Mockito.doReturn;
import static org.mockito.Mockito.mock;
import static org.mockito.Mockito.reset;
import static org.mockito.Mockito.spy;
import static org.mockito.Mockito.times;
import static org.mockito.Mockito.verify;
import java.io.File;
import java.io.IOException;
import java.nio.ByteBuffer;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Map;
import org.apache.commons.logging.Log; import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory; import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.conf.Configuration;
@ -52,9 +31,9 @@ import org.apache.hadoop.security.Credentials;
import org.apache.hadoop.security.SecurityUtil; import org.apache.hadoop.security.SecurityUtil;
import org.apache.hadoop.security.UserGroupInformation; import org.apache.hadoop.security.UserGroupInformation;
import org.apache.hadoop.security.UserGroupInformation.AuthenticationMethod; import org.apache.hadoop.security.UserGroupInformation.AuthenticationMethod;
import org.apache.hadoop.test.GenericTestUtils;
import org.apache.hadoop.yarn.MockApps; import org.apache.hadoop.yarn.MockApps;
import org.apache.hadoop.yarn.api.ApplicationConstants.Environment; import org.apache.hadoop.yarn.api.ApplicationConstants.Environment;
import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
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.api.records.ApplicationSubmissionContext; import org.apache.hadoop.yarn.api.records.ApplicationSubmissionContext;
@ -89,7 +68,12 @@ 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;
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType; import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptEventType;
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptImpl; import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt.RMAppAttemptImpl;
import org.apache.hadoop.yarn.server.resourcemanager.rmapp.attempt
.RMAppAttemptState;
import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.ContainerAllocationExpirer; import org.apache.hadoop.yarn.server.resourcemanager.rmcontainer.ContainerAllocationExpirer;
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler; import org.apache.hadoop.yarn.server.resourcemanager.scheduler.ResourceScheduler;
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.YarnScheduler; import org.apache.hadoop.yarn.server.resourcemanager.scheduler.YarnScheduler;
import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.AppRemovedSchedulerEvent; import org.apache.hadoop.yarn.server.resourcemanager.scheduler.event.AppRemovedSchedulerEvent;
@ -111,6 +95,29 @@ import org.junit.runners.Parameterized;
import org.mockito.ArgumentCaptor; import org.mockito.ArgumentCaptor;
import org.mockito.Matchers; import org.mockito.Matchers;
import java.io.File;
import java.io.IOException;
import java.nio.ByteBuffer;
import java.util.ArrayList;
import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
import static junit.framework.TestCase.assertTrue;
import static org.junit.Assert.assertEquals;
import static org.mockito.Matchers.any;
import static org.mockito.Matchers.anyLong;
import static org.mockito.Matchers.eq;
import static org.mockito.Mockito.doReturn;
import static org.mockito.Mockito.mock;
import static org.mockito.Mockito.reset;
import static org.mockito.Mockito.spy;
import static org.mockito.Mockito.times;
import static org.mockito.Mockito.verify;
@RunWith(value = Parameterized.class) @RunWith(value = Parameterized.class)
public class TestRMAppTransitions { public class TestRMAppTransitions {
@ -128,6 +135,8 @@ public class TestRMAppTransitions {
private SystemMetricsPublisher publisher; private SystemMetricsPublisher publisher;
private YarnScheduler scheduler; private YarnScheduler scheduler;
private TestSchedulerEventDispatcher schedulerDispatcher; private TestSchedulerEventDispatcher schedulerDispatcher;
private TestApplicationManagerEventDispatcher appManagerDispatcher;
private long testCaseStartTime;
// ignore all the RM application attempt events // ignore all the RM application attempt events
private static final class TestApplicationAttemptEventDispatcher implements private static final class TestApplicationAttemptEventDispatcher implements
@ -181,8 +190,11 @@ public class TestRMAppTransitions {
// ResourceManager.java // ResourceManager.java
private static final class TestApplicationManagerEventDispatcher implements private static final class TestApplicationManagerEventDispatcher implements
EventHandler<RMAppManagerEvent> { EventHandler<RMAppManagerEvent> {
List<RMAppManagerEvent> events = Lists.newArrayList();
@Override @Override
public void handle(RMAppManagerEvent event) { public void handle(RMAppManagerEvent event) {
LOG.info("Handling app manager event: " + event);
events.add(event);
} }
} }
@ -243,7 +255,7 @@ public class TestRMAppTransitions {
ResourceScheduler resourceScheduler = mock(ResourceScheduler.class); ResourceScheduler resourceScheduler = mock(ResourceScheduler.class);
doReturn(null).when(resourceScheduler) doReturn(null).when(resourceScheduler)
.getAppResourceUsageReport((ApplicationAttemptId)Matchers.any()); .getAppResourceUsageReport(Matchers.any());
doReturn(resourceScheduler).when(rmContext).getScheduler(); doReturn(resourceScheduler).when(rmContext).getScheduler();
doReturn(mock(RMTimelineCollectorManager.class)).when(rmContext) doReturn(mock(RMTimelineCollectorManager.class)).when(rmContext)
@ -255,8 +267,10 @@ public class TestRMAppTransitions {
rmDispatcher.register(RMAppEventType.class, rmDispatcher.register(RMAppEventType.class,
new TestApplicationEventDispatcher(rmContext)); new TestApplicationEventDispatcher(rmContext));
appManagerDispatcher = new
TestApplicationManagerEventDispatcher();
rmDispatcher.register(RMAppManagerEventType.class, rmDispatcher.register(RMAppManagerEventType.class,
new TestApplicationManagerEventDispatcher()); appManagerDispatcher);
schedulerDispatcher = new TestSchedulerEventDispatcher(); schedulerDispatcher = new TestSchedulerEventDispatcher();
rmDispatcher.register(SchedulerEventType.class, rmDispatcher.register(SchedulerEventType.class,
@ -264,6 +278,7 @@ public class TestRMAppTransitions {
rmDispatcher.init(conf); rmDispatcher.init(conf);
rmDispatcher.start(); rmDispatcher.start();
testCaseStartTime = System.currentTimeMillis();
} }
private ByteBuffer getTokens() throws IOException { private ByteBuffer getTokens() throws IOException {
@ -332,7 +347,7 @@ public class TestRMAppTransitions {
ContainerRetryContext containerRetryContext = ContainerRetryContext ContainerRetryContext containerRetryContext = ContainerRetryContext
.newInstance( .newInstance(
ContainerRetryPolicy.RETRY_ON_SPECIFIC_ERROR_CODES, ContainerRetryPolicy.RETRY_ON_SPECIFIC_ERROR_CODES,
new HashSet<>(Arrays.asList(Integer.valueOf(111))), 0, 0); new HashSet<>(Arrays.asList(111)), 0, 0);
return containerRetryContext; return containerRetryContext;
} }
@ -424,17 +439,17 @@ public class TestRMAppTransitions {
name, application.getName()); name, application.getName());
Assert.assertEquals("application finish time is not 0 and should be", Assert.assertEquals("application finish time is not 0 and should be",
0, application.getFinishTime()); 0, application.getFinishTime());
Assert.assertEquals("application tracking url is not correct", Assert.assertNull("application tracking url is not correct",
null, application.getTrackingUrl()); application.getTrackingUrl());
StringBuilder diag = application.getDiagnostics(); StringBuilder diag = application.getDiagnostics();
Assert.assertEquals("application diagnostics is not correct", Assert.assertEquals("application diagnostics is not correct",
0, diag.length()); 0, diag.length());
} }
// test to make sure times are set when app finishes // test to make sure times are set when app finishes
private static void assertStartTimeSet(RMApp application) { private void assertStartTimeSet(RMApp application) {
Assert.assertTrue("application start time is not greater than 0", Assert.assertTrue("application start time is before test case start time",
application.getStartTime() > 0); application.getStartTime() >= testCaseStartTime);
Assert.assertTrue("application start time is before currentTime", Assert.assertTrue("application start time is before currentTime",
application.getStartTime() <= System.currentTimeMillis()); application.getStartTime() <= System.currentTimeMillis());
} }
@ -452,8 +467,6 @@ public class TestRMAppTransitions {
// test to make sure times are set when app finishes // test to make sure times are set when app finishes
private void assertTimesAtFinish(RMApp application) { private void assertTimesAtFinish(RMApp application) {
assertStartTimeSet(application); assertStartTimeSet(application);
Assert.assertTrue("application finish time is not greater than 0",
(application.getFinishTime() > 0));
Assert.assertTrue("application finish time is not >= start time", Assert.assertTrue("application finish time is not >= start time",
(application.getFinishTime() >= application.getStartTime())); (application.getFinishTime() >= application.getStartTime()));
} }
@ -537,8 +550,10 @@ public class TestRMAppTransitions {
RMApp application = createNewTestApp(submissionContext); RMApp application = createNewTestApp(submissionContext);
// NEW => SUBMITTED event RMAppEventType.RECOVER // NEW => SUBMITTED event RMAppEventType.RECOVER
RMState state = new RMState(); RMState state = new RMState();
long startTime = testCaseStartTime + 1;
ApplicationStateData appState = ApplicationStateData appState =
ApplicationStateData.newInstance(123, 123, null, "user", null); ApplicationStateData.newInstance(testCaseStartTime, startTime, null,
"user", null);
state.getApplicationState().put(application.getApplicationId(), appState); state.getApplicationState().put(application.getApplicationId(), appState);
RMAppEvent event = RMAppEvent event =
new RMAppRecoverEvent(application.getApplicationId(), state); new RMAppRecoverEvent(application.getApplicationId(), state);
@ -590,14 +605,21 @@ public class TestRMAppTransitions {
} }
protected RMApp testCreateAppFinishing( protected RMApp testCreateAppFinishing(
ApplicationSubmissionContext submissionContext) throws IOException { ApplicationSubmissionContext submissionContext) throws Exception {
// unmanaged AMs don't use the FINISHING state // unmanaged AMs don't use the FINISHING state
assert submissionContext == null || !submissionContext.getUnmanagedAM(); assert submissionContext == null || !submissionContext.getUnmanagedAM();
RMApp application = testCreateAppFinalSaving(submissionContext); RMApp application = testCreateAppFinalSaving(submissionContext);
Assert.assertNotNull("app shouldn't be null", application);
// FINAL_SAVING => FINISHING event RMAppEventType.APP_UPDATED // FINAL_SAVING => FINISHING event RMAppEventType.APP_UPDATED
RMAppEvent appUpdated = RMAppEvent appUpdated =
new RMAppEvent(application.getApplicationId(), RMAppEventType.APP_UPDATE_SAVED); new RMAppEvent(application.getApplicationId(), RMAppEventType.APP_UPDATE_SAVED);
application.handle(appUpdated); application.handle(appUpdated);
GenericTestUtils.waitFor(() -> {
RMAppAttempt appAttempt = application.getCurrentAppAttempt();
return appAttempt != null &&
RMAppAttemptState.SUBMITTED.equals(appAttempt.getState());
}, 10, 80 * 1000);
assertAppState(RMAppState.FINISHING, application); assertAppState(RMAppState.FINISHING, application);
assertTimesAtFinish(application); assertTimesAtFinish(application);
return application; return application;
@ -605,7 +627,7 @@ public class TestRMAppTransitions {
protected RMApp testCreateAppFinished( protected RMApp testCreateAppFinished(
ApplicationSubmissionContext submissionContext, ApplicationSubmissionContext submissionContext,
String diagnostics) throws IOException { String diagnostics) throws Exception {
// unmanaged AMs don't use the FINISHING state // unmanaged AMs don't use the FINISHING state
RMApp application = null; RMApp application = null;
if (submissionContext != null && submissionContext.getUnmanagedAM()) { if (submissionContext != null && submissionContext.getUnmanagedAM()) {
@ -613,10 +635,17 @@ public class TestRMAppTransitions {
} else { } else {
application = testCreateAppFinishing(submissionContext); application = testCreateAppFinishing(submissionContext);
} }
verifyAppBeforeFinishEvent(application);
// RUNNING/FINISHING => FINISHED event RMAppEventType.ATTEMPT_FINISHED // RUNNING/FINISHING => FINISHED event RMAppEventType.ATTEMPT_FINISHED
RMAppEvent finishedEvent = new RMAppEvent(application.getApplicationId(), RMAppEvent finishedEvent = new RMAppEvent(application.getApplicationId(),
RMAppEventType.ATTEMPT_FINISHED, diagnostics); RMAppEventType.ATTEMPT_FINISHED, diagnostics);
application.handle(finishedEvent); application.handle(finishedEvent);
//only run this verification if we created a finishing app
if (submissionContext == null) {
verifyAppAfterFinishEvent(application);
}
assertAppState(RMAppState.FINISHED, application); assertAppState(RMAppState.FINISHED, application);
assertTimesAtFinish(application); assertTimesAtFinish(application);
// finished without a proper unregister implies failed // finished without a proper unregister implies failed
@ -627,7 +656,7 @@ public class TestRMAppTransitions {
} }
@Test @Test
public void testUnmanagedApp() throws IOException { public void testUnmanagedApp() throws Exception {
ApplicationSubmissionContext subContext = new ApplicationSubmissionContextPBImpl(); ApplicationSubmissionContext subContext = new ApplicationSubmissionContextPBImpl();
subContext.setUnmanagedAM(true); subContext.setUnmanagedAM(true);
@ -659,7 +688,7 @@ public class TestRMAppTransitions {
} }
@Test @Test
public void testAppSuccessPath() throws IOException { public void testAppSuccessPath() throws Exception {
LOG.info("--- START: testAppSuccessPath ---"); LOG.info("--- START: testAppSuccessPath ---");
final String diagMsg = "some diagnostics"; final String diagMsg = "some diagnostics";
RMApp application = testCreateAppFinished(null, diagMsg); RMApp application = testCreateAppFinished(null, diagMsg);
@ -695,7 +724,7 @@ public class TestRMAppTransitions {
assertKilled(application); assertKilled(application);
assertAppFinalStateNotSaved(application); assertAppFinalStateNotSaved(application);
verifyApplicationFinished(RMAppState.KILLED); verifyApplicationFinished(RMAppState.KILLED);
verifyAppRemovedSchedulerEvent(RMAppState.KILLED); verifyAppRemovedSchedulerEvent(application, RMAppState.KILLED);
verifyRMAppFieldsForFinalTransitions(application); verifyRMAppFieldsForFinalTransitions(application);
} }
@ -754,7 +783,7 @@ public class TestRMAppTransitions {
sendAppUpdateSavedEvent(application); sendAppUpdateSavedEvent(application);
assertKilled(application); assertKilled(application);
verifyApplicationFinished(RMAppState.KILLED); verifyApplicationFinished(RMAppState.KILLED);
verifyAppRemovedSchedulerEvent(RMAppState.KILLED); verifyAppRemovedSchedulerEvent(application, RMAppState.KILLED);
verifyRMAppFieldsForFinalTransitions(application); verifyRMAppFieldsForFinalTransitions(application);
} }
@ -830,7 +859,7 @@ public class TestRMAppTransitions {
assertKilled(application); assertKilled(application);
assertAppFinalStateSaved(application); assertAppFinalStateSaved(application);
verifyApplicationFinished(RMAppState.KILLED); verifyApplicationFinished(RMAppState.KILLED);
verifyAppRemovedSchedulerEvent(RMAppState.KILLED); verifyAppRemovedSchedulerEvent(application, RMAppState.KILLED);
verifyRMAppFieldsForFinalTransitions(application); verifyRMAppFieldsForFinalTransitions(application);
} }
@ -894,7 +923,7 @@ public class TestRMAppTransitions {
assertKilled(application); assertKilled(application);
assertAppFinalStateSaved(application); assertAppFinalStateSaved(application);
verifyApplicationFinished(RMAppState.KILLED); verifyApplicationFinished(RMAppState.KILLED);
verifyAppRemovedSchedulerEvent(RMAppState.KILLED); verifyAppRemovedSchedulerEvent(application, RMAppState.KILLED);
verifyRMAppFieldsForFinalTransitions(application); verifyRMAppFieldsForFinalTransitions(application);
} }
@ -918,7 +947,7 @@ public class TestRMAppTransitions {
assertKilled(application); assertKilled(application);
assertAppFinalStateSaved(application); assertAppFinalStateSaved(application);
verifyApplicationFinished(RMAppState.KILLED); verifyApplicationFinished(RMAppState.KILLED);
verifyAppRemovedSchedulerEvent(RMAppState.KILLED); verifyAppRemovedSchedulerEvent(application, RMAppState.KILLED);
} }
@Test @Test
@ -942,7 +971,7 @@ public class TestRMAppTransitions {
sendAppUpdateSavedEvent(application); sendAppUpdateSavedEvent(application);
assertKilled(application); assertKilled(application);
verifyApplicationFinished(RMAppState.KILLED); verifyApplicationFinished(RMAppState.KILLED);
verifyAppRemovedSchedulerEvent(RMAppState.KILLED); verifyAppRemovedSchedulerEvent(application, RMAppState.KILLED);
verifyRMAppFieldsForFinalTransitions(application); verifyRMAppFieldsForFinalTransitions(application);
} }
@ -1005,7 +1034,7 @@ public class TestRMAppTransitions {
} }
@Test @Test
public void testAppAtFinishingIgnoreKill() throws IOException { public void testAppAtFinishingIgnoreKill() throws Exception {
LOG.info("--- START: testAppAtFinishingIgnoreKill ---"); LOG.info("--- START: testAppAtFinishingIgnoreKill ---");
RMApp application = testCreateAppFinishing(null); RMApp application = testCreateAppFinishing(null);
@ -1047,7 +1076,7 @@ public class TestRMAppTransitions {
} }
@Test @Test
public void testAppFinishedFinished() throws IOException { public void testAppFinishedFinished() throws Exception {
LOG.info("--- START: testAppFinishedFinished ---"); LOG.info("--- START: testAppFinishedFinished ---");
RMApp application = testCreateAppFinished(null, ""); RMApp application = testCreateAppFinished(null, "");
@ -1063,6 +1092,7 @@ public class TestRMAppTransitions {
Assert.assertEquals("application diagnostics is not correct", Assert.assertEquals("application diagnostics is not correct",
"", diag.toString()); "", diag.toString());
verifyApplicationFinished(RMAppState.FINISHED); verifyApplicationFinished(RMAppState.FINISHED);
verifyAppRemovedSchedulerEvent(application, RMAppState.FINISHED);
verifyRMAppFieldsForFinalTransitions(application); verifyRMAppFieldsForFinalTransitions(application);
} }
@ -1152,7 +1182,7 @@ public class TestRMAppTransitions {
} }
@Test (timeout = 30000) @Test (timeout = 30000)
public void testAppStartAfterKilled() throws IOException { public void testAppStartAfterKilled() {
LOG.info("--- START: testAppStartAfterKilled ---"); LOG.info("--- START: testAppStartAfterKilled ---");
ApplicationId applicationId = MockApps.newAppID(appId++); ApplicationId applicationId = MockApps.newAppID(appId++);
@ -1162,8 +1192,8 @@ public class TestRMAppTransitions {
@Override @Override
protected void onInvalidStateTransition(RMAppEventType rmAppEventType, protected void onInvalidStateTransition(RMAppEventType rmAppEventType,
RMAppState state) { RMAppState state) {
Assert.assertTrue("RMAppImpl: can't handle " + rmAppEventType Assert.fail("RMAppImpl: can't handle " + rmAppEventType
+ " at state " + state, false); + " at state " + state);
} }
}; };
@ -1200,8 +1230,7 @@ public class TestRMAppTransitions {
} }
public void testRecoverApplication(ApplicationStateData appState, public void testRecoverApplication(ApplicationStateData appState,
RMState rmState) RMState rmState) {
throws Exception {
ApplicationSubmissionContext submissionContext = ApplicationSubmissionContext submissionContext =
appState.getApplicationSubmissionContext(); appState.getApplicationSubmissionContext();
RMAppImpl application = RMAppImpl application =
@ -1257,6 +1286,30 @@ public class TestRMAppTransitions {
+ "/")); + "/"));
} }
private void verifyAppBeforeFinishEvent(RMApp app) {
assertEquals(0L, ((RMAppImpl) app).getLogAggregationStartTime());
//RMAppEventType.APP_UPDATE_SAVED sets the finish time
assertTrue("App manager events should not be received!",
appManagerDispatcher.events.isEmpty());
}
private void verifyAppAfterFinishEvent(RMApp app) {
assertTrue(
testCaseStartTime < ((RMAppImpl) app).getLogAggregationStartTime());
assertAppState(RMAppState.FINISHED, app);
verifyAppCompletedEvent(app);
verifyAppRemovedSchedulerEvent(app, RMAppState.FINISHED);
}
private void verifyAppCompletedEvent(RMApp app) {
assertEquals(1, appManagerDispatcher.events.size());
RMAppManagerEvent rmAppManagerEvent = appManagerDispatcher.events.get(0);
assertEquals(RMAppManagerEventType.APP_COMPLETED,
rmAppManagerEvent.getType());
assertEquals(app.getApplicationId().getId(),
rmAppManagerEvent.getApplicationId().getId());
}
private void verifyApplicationFinished(RMAppState state) { private void verifyApplicationFinished(RMAppState state) {
ArgumentCaptor<RMAppState> finalState = ArgumentCaptor<RMAppState> finalState =
ArgumentCaptor.forClass(RMAppState.class); ArgumentCaptor.forClass(RMAppState.class);
@ -1268,14 +1321,16 @@ public class TestRMAppTransitions {
Assert.assertEquals(state, finalState.getValue()); Assert.assertEquals(state, finalState.getValue());
} }
private void verifyAppRemovedSchedulerEvent(RMAppState finalState) { private void verifyAppRemovedSchedulerEvent(RMApp app,
Assert.assertEquals(SchedulerEventType.APP_REMOVED, RMAppState finalState) {
schedulerDispatcher.lastSchedulerEvent.getType()); SchedulerEvent lastEvent = schedulerDispatcher.lastSchedulerEvent;
if(schedulerDispatcher.lastSchedulerEvent instanceof Assert.assertEquals(SchedulerEventType.APP_REMOVED, lastEvent.getType());
AppRemovedSchedulerEvent) { if (lastEvent instanceof AppRemovedSchedulerEvent) {
AppRemovedSchedulerEvent appRemovedEvent = AppRemovedSchedulerEvent appRemovedEvent =
(AppRemovedSchedulerEvent) schedulerDispatcher.lastSchedulerEvent; (AppRemovedSchedulerEvent) lastEvent;
Assert.assertEquals(finalState, appRemovedEvent.getFinalState()); Assert.assertEquals(finalState, appRemovedEvent.getFinalState());
Assert.assertEquals(app.getApplicationId().getId(),
appRemovedEvent.getApplicationID().getId());
} }
} }