Revert "MAPREDUCE-6898. TestKill.testKillTask is flaky. Contributed by Peter Bacsko"

This reverts commit e78542f4be.
This commit is contained in:
Jason Lowe 2017-06-16 14:48:51 -05:00
parent e78542f4be
commit 9ef2743fa1

View File

@ -21,15 +21,11 @@
import java.util.Iterator; import java.util.Iterator;
import java.util.Map; import java.util.Map;
import java.util.concurrent.CountDownLatch; import java.util.concurrent.CountDownLatch;
import java.util.concurrent.TimeUnit;
import org.apache.hadoop.service.Service; import org.apache.hadoop.service.Service;
import org.junit.Assert; import org.junit.Assert;
import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.conf.Configuration; import org.apache.hadoop.conf.Configuration;
import org.apache.hadoop.mapreduce.jobhistory.EventType;
import org.apache.hadoop.mapreduce.jobhistory.TaskStartedEvent;
import org.apache.hadoop.mapreduce.v2.api.records.JobId; import org.apache.hadoop.mapreduce.v2.api.records.JobId;
import org.apache.hadoop.mapreduce.v2.api.records.JobState; import org.apache.hadoop.mapreduce.v2.api.records.JobState;
import org.apache.hadoop.mapreduce.v2.api.records.TaskAttemptId; import org.apache.hadoop.mapreduce.v2.api.records.TaskAttemptId;
@ -104,34 +100,12 @@ public void testKillJob() throws Exception {
@Test @Test
public void testKillTask() throws Exception { public void testKillTask() throws Exception {
final CountDownLatch latch = new CountDownLatch(1); final CountDownLatch latch = new CountDownLatch(1);
final CountDownLatch jobRunning = new CountDownLatch(1); MRApp app = new BlockingMRApp(2, 0, latch);
final Dispatcher dispatcher = new AsyncDispatcher() {
protected void dispatch(Event event) {
// We have to wait until the internal state is RUNNING and not SETUP
// because it can cause a transition failure. If the dispatchable
// event is TASK_STARTED, we can continue because the job must be in
// RUNNING at that point.
if (event.getType() == EventType.TASK_STARTED) {
jobRunning.countDown();
}
super.dispatch(event);
}
};
MRApp app = new BlockingMRApp(2, 0, latch) {
@Override
public Dispatcher createDispatcher() {
return dispatcher;
}
};
//this will start the job but job won't complete as Task is blocked //this will start the job but job won't complete as Task is blocked
Job job = app.submit(new Configuration()); Job job = app.submit(new Configuration());
// wait until the job transitions to JobInternalState.RUNNING //wait and vailidate for Job to become RUNNING
jobRunning.await(10, TimeUnit.SECONDS); app.waitForInternalState((JobImpl) job, JobStateInternal.RUNNING);
Map<TaskId,Task> tasks = job.getTasks(); Map<TaskId,Task> tasks = job.getTasks();
Assert.assertEquals("No of tasks is not correct", 2, Assert.assertEquals("No of tasks is not correct", 2,
tasks.size()); tasks.size());