|
@@ -41,7 +41,6 @@ import org.apache.hadoop.conf.Configuration;
|
|
import org.apache.hadoop.fs.FSDataInputStream;
|
|
import org.apache.hadoop.fs.FSDataInputStream;
|
|
import org.apache.hadoop.fs.FileSystem;
|
|
import org.apache.hadoop.fs.FileSystem;
|
|
import org.apache.hadoop.fs.Path;
|
|
import org.apache.hadoop.fs.Path;
|
|
-import org.apache.hadoop.http.HttpConfig;
|
|
|
|
import org.apache.hadoop.mapred.FileOutputCommitter;
|
|
import org.apache.hadoop.mapred.FileOutputCommitter;
|
|
import org.apache.hadoop.mapred.JobConf;
|
|
import org.apache.hadoop.mapred.JobConf;
|
|
import org.apache.hadoop.mapred.LocalContainerLauncher;
|
|
import org.apache.hadoop.mapred.LocalContainerLauncher;
|
|
@@ -200,6 +199,7 @@ public class MRAppMaster extends CompositeService {
|
|
new JobTokenSecretManager();
|
|
new JobTokenSecretManager();
|
|
private JobId jobId;
|
|
private JobId jobId;
|
|
private boolean newApiCommitter;
|
|
private boolean newApiCommitter;
|
|
|
|
+ private ClassLoader jobClassLoader;
|
|
private OutputCommitter committer;
|
|
private OutputCommitter committer;
|
|
private JobEventDispatcher jobEventDispatcher;
|
|
private JobEventDispatcher jobEventDispatcher;
|
|
private JobHistoryEventHandler jobHistoryEventHandler;
|
|
private JobHistoryEventHandler jobHistoryEventHandler;
|
|
@@ -250,6 +250,9 @@ public class MRAppMaster extends CompositeService {
|
|
|
|
|
|
@Override
|
|
@Override
|
|
protected void serviceInit(final Configuration conf) throws Exception {
|
|
protected void serviceInit(final Configuration conf) throws Exception {
|
|
|
|
+ // create the job classloader if enabled
|
|
|
|
+ createJobClassLoader(conf);
|
|
|
|
+
|
|
conf.setBoolean(Dispatcher.DISPATCHER_EXIT_ON_ERROR_KEY, true);
|
|
conf.setBoolean(Dispatcher.DISPATCHER_EXIT_ON_ERROR_KEY, true);
|
|
|
|
|
|
initJobCredentialsAndUGI(conf);
|
|
initJobCredentialsAndUGI(conf);
|
|
@@ -387,8 +390,13 @@ public class MRAppMaster extends CompositeService {
|
|
addIfService(committerEventHandler);
|
|
addIfService(committerEventHandler);
|
|
|
|
|
|
//policy handling preemption requests from RM
|
|
//policy handling preemption requests from RM
|
|
- preemptionPolicy = createPreemptionPolicy(conf);
|
|
|
|
- preemptionPolicy.init(context);
|
|
|
|
|
|
+ callWithJobClassLoader(conf, new Action<Void>() {
|
|
|
|
+ public Void call(Configuration conf) {
|
|
|
|
+ preemptionPolicy = createPreemptionPolicy(conf);
|
|
|
|
+ preemptionPolicy.init(context);
|
|
|
|
+ return null;
|
|
|
|
+ }
|
|
|
|
+ });
|
|
|
|
|
|
//service to handle requests to TaskUmbilicalProtocol
|
|
//service to handle requests to TaskUmbilicalProtocol
|
|
taskAttemptListener = createTaskAttemptListener(context, preemptionPolicy);
|
|
taskAttemptListener = createTaskAttemptListener(context, preemptionPolicy);
|
|
@@ -453,33 +461,37 @@ public class MRAppMaster extends CompositeService {
|
|
}
|
|
}
|
|
|
|
|
|
private OutputCommitter createOutputCommitter(Configuration conf) {
|
|
private OutputCommitter createOutputCommitter(Configuration conf) {
|
|
- OutputCommitter committer = null;
|
|
|
|
-
|
|
|
|
- LOG.info("OutputCommitter set in config "
|
|
|
|
- + conf.get("mapred.output.committer.class"));
|
|
|
|
-
|
|
|
|
- if (newApiCommitter) {
|
|
|
|
- org.apache.hadoop.mapreduce.v2.api.records.TaskId taskID = MRBuilderUtils
|
|
|
|
- .newTaskId(jobId, 0, TaskType.MAP);
|
|
|
|
- org.apache.hadoop.mapreduce.v2.api.records.TaskAttemptId attemptID = MRBuilderUtils
|
|
|
|
- .newTaskAttemptId(taskID, 0);
|
|
|
|
- TaskAttemptContext taskContext = new TaskAttemptContextImpl(conf,
|
|
|
|
- TypeConverter.fromYarn(attemptID));
|
|
|
|
- OutputFormat outputFormat;
|
|
|
|
- try {
|
|
|
|
- outputFormat = ReflectionUtils.newInstance(taskContext
|
|
|
|
- .getOutputFormatClass(), conf);
|
|
|
|
- committer = outputFormat.getOutputCommitter(taskContext);
|
|
|
|
- } catch (Exception e) {
|
|
|
|
- throw new YarnRuntimeException(e);
|
|
|
|
|
|
+ return callWithJobClassLoader(conf, new Action<OutputCommitter>() {
|
|
|
|
+ public OutputCommitter call(Configuration conf) {
|
|
|
|
+ OutputCommitter committer = null;
|
|
|
|
+
|
|
|
|
+ LOG.info("OutputCommitter set in config "
|
|
|
|
+ + conf.get("mapred.output.committer.class"));
|
|
|
|
+
|
|
|
|
+ if (newApiCommitter) {
|
|
|
|
+ org.apache.hadoop.mapreduce.v2.api.records.TaskId taskID =
|
|
|
|
+ MRBuilderUtils.newTaskId(jobId, 0, TaskType.MAP);
|
|
|
|
+ org.apache.hadoop.mapreduce.v2.api.records.TaskAttemptId attemptID =
|
|
|
|
+ MRBuilderUtils.newTaskAttemptId(taskID, 0);
|
|
|
|
+ TaskAttemptContext taskContext = new TaskAttemptContextImpl(conf,
|
|
|
|
+ TypeConverter.fromYarn(attemptID));
|
|
|
|
+ OutputFormat outputFormat;
|
|
|
|
+ try {
|
|
|
|
+ outputFormat = ReflectionUtils.newInstance(taskContext
|
|
|
|
+ .getOutputFormatClass(), conf);
|
|
|
|
+ committer = outputFormat.getOutputCommitter(taskContext);
|
|
|
|
+ } catch (Exception e) {
|
|
|
|
+ throw new YarnRuntimeException(e);
|
|
|
|
+ }
|
|
|
|
+ } else {
|
|
|
|
+ committer = ReflectionUtils.newInstance(conf.getClass(
|
|
|
|
+ "mapred.output.committer.class", FileOutputCommitter.class,
|
|
|
|
+ org.apache.hadoop.mapred.OutputCommitter.class), conf);
|
|
|
|
+ }
|
|
|
|
+ LOG.info("OutputCommitter is " + committer.getClass().getName());
|
|
|
|
+ return committer;
|
|
}
|
|
}
|
|
- } else {
|
|
|
|
- committer = ReflectionUtils.newInstance(conf.getClass(
|
|
|
|
- "mapred.output.committer.class", FileOutputCommitter.class,
|
|
|
|
- org.apache.hadoop.mapred.OutputCommitter.class), conf);
|
|
|
|
- }
|
|
|
|
- LOG.info("OutputCommitter is " + committer.getClass().getName());
|
|
|
|
- return committer;
|
|
|
|
|
|
+ });
|
|
}
|
|
}
|
|
|
|
|
|
protected AMPreemptionPolicy createPreemptionPolicy(Configuration conf) {
|
|
protected AMPreemptionPolicy createPreemptionPolicy(Configuration conf) {
|
|
@@ -667,38 +679,42 @@ public class MRAppMaster extends CompositeService {
|
|
return new StagingDirCleaningService();
|
|
return new StagingDirCleaningService();
|
|
}
|
|
}
|
|
|
|
|
|
- protected Speculator createSpeculator(Configuration conf, AppContext context) {
|
|
|
|
- Class<? extends Speculator> speculatorClass;
|
|
|
|
-
|
|
|
|
- try {
|
|
|
|
- speculatorClass
|
|
|
|
- // "yarn.mapreduce.job.speculator.class"
|
|
|
|
- = conf.getClass(MRJobConfig.MR_AM_JOB_SPECULATOR,
|
|
|
|
- DefaultSpeculator.class,
|
|
|
|
- Speculator.class);
|
|
|
|
- Constructor<? extends Speculator> speculatorConstructor
|
|
|
|
- = speculatorClass.getConstructor
|
|
|
|
- (Configuration.class, AppContext.class);
|
|
|
|
- Speculator result = speculatorConstructor.newInstance(conf, context);
|
|
|
|
-
|
|
|
|
- return result;
|
|
|
|
- } catch (InstantiationException ex) {
|
|
|
|
- LOG.error("Can't make a speculator -- check "
|
|
|
|
- + MRJobConfig.MR_AM_JOB_SPECULATOR, ex);
|
|
|
|
- throw new YarnRuntimeException(ex);
|
|
|
|
- } catch (IllegalAccessException ex) {
|
|
|
|
- LOG.error("Can't make a speculator -- check "
|
|
|
|
- + MRJobConfig.MR_AM_JOB_SPECULATOR, ex);
|
|
|
|
- throw new YarnRuntimeException(ex);
|
|
|
|
- } catch (InvocationTargetException ex) {
|
|
|
|
- LOG.error("Can't make a speculator -- check "
|
|
|
|
- + MRJobConfig.MR_AM_JOB_SPECULATOR, ex);
|
|
|
|
- throw new YarnRuntimeException(ex);
|
|
|
|
- } catch (NoSuchMethodException ex) {
|
|
|
|
- LOG.error("Can't make a speculator -- check "
|
|
|
|
- + MRJobConfig.MR_AM_JOB_SPECULATOR, ex);
|
|
|
|
- throw new YarnRuntimeException(ex);
|
|
|
|
- }
|
|
|
|
|
|
+ protected Speculator createSpeculator(Configuration conf,
|
|
|
|
+ final AppContext context) {
|
|
|
|
+ return callWithJobClassLoader(conf, new Action<Speculator>() {
|
|
|
|
+ public Speculator call(Configuration conf) {
|
|
|
|
+ Class<? extends Speculator> speculatorClass;
|
|
|
|
+ try {
|
|
|
|
+ speculatorClass
|
|
|
|
+ // "yarn.mapreduce.job.speculator.class"
|
|
|
|
+ = conf.getClass(MRJobConfig.MR_AM_JOB_SPECULATOR,
|
|
|
|
+ DefaultSpeculator.class,
|
|
|
|
+ Speculator.class);
|
|
|
|
+ Constructor<? extends Speculator> speculatorConstructor
|
|
|
|
+ = speculatorClass.getConstructor
|
|
|
|
+ (Configuration.class, AppContext.class);
|
|
|
|
+ Speculator result = speculatorConstructor.newInstance(conf, context);
|
|
|
|
+
|
|
|
|
+ return result;
|
|
|
|
+ } catch (InstantiationException ex) {
|
|
|
|
+ LOG.error("Can't make a speculator -- check "
|
|
|
|
+ + MRJobConfig.MR_AM_JOB_SPECULATOR, ex);
|
|
|
|
+ throw new YarnRuntimeException(ex);
|
|
|
|
+ } catch (IllegalAccessException ex) {
|
|
|
|
+ LOG.error("Can't make a speculator -- check "
|
|
|
|
+ + MRJobConfig.MR_AM_JOB_SPECULATOR, ex);
|
|
|
|
+ throw new YarnRuntimeException(ex);
|
|
|
|
+ } catch (InvocationTargetException ex) {
|
|
|
|
+ LOG.error("Can't make a speculator -- check "
|
|
|
|
+ + MRJobConfig.MR_AM_JOB_SPECULATOR, ex);
|
|
|
|
+ throw new YarnRuntimeException(ex);
|
|
|
|
+ } catch (NoSuchMethodException ex) {
|
|
|
|
+ LOG.error("Can't make a speculator -- check "
|
|
|
|
+ + MRJobConfig.MR_AM_JOB_SPECULATOR, ex);
|
|
|
|
+ throw new YarnRuntimeException(ex);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ });
|
|
}
|
|
}
|
|
|
|
|
|
protected TaskAttemptListener createTaskAttemptListener(AppContext context,
|
|
protected TaskAttemptListener createTaskAttemptListener(AppContext context,
|
|
@@ -712,7 +728,7 @@ public class MRAppMaster extends CompositeService {
|
|
protected EventHandler<CommitterEvent> createCommitterEventHandler(
|
|
protected EventHandler<CommitterEvent> createCommitterEventHandler(
|
|
AppContext context, OutputCommitter committer) {
|
|
AppContext context, OutputCommitter committer) {
|
|
return new CommitterEventHandler(context, committer,
|
|
return new CommitterEventHandler(context, committer,
|
|
- getRMHeartbeatHandler());
|
|
|
|
|
|
+ getRMHeartbeatHandler(), jobClassLoader);
|
|
}
|
|
}
|
|
|
|
|
|
protected ContainerAllocator createContainerAllocator(
|
|
protected ContainerAllocator createContainerAllocator(
|
|
@@ -1083,8 +1099,8 @@ public class MRAppMaster extends CompositeService {
|
|
//start all the components
|
|
//start all the components
|
|
super.serviceStart();
|
|
super.serviceStart();
|
|
|
|
|
|
- // set job classloader if configured
|
|
|
|
- MRApps.setJobClassLoader(getConfig());
|
|
|
|
|
|
+ // finally set the job classloader
|
|
|
|
+ MRApps.setClassLoader(jobClassLoader, getConfig());
|
|
|
|
|
|
if (initFailed) {
|
|
if (initFailed) {
|
|
JobEvent initFailedEvent = new JobEvent(job.getID(), JobEventType.JOB_INIT_FAILED);
|
|
JobEvent initFailedEvent = new JobEvent(job.getID(), JobEventType.JOB_INIT_FAILED);
|
|
@@ -1101,19 +1117,24 @@ public class MRAppMaster extends CompositeService {
|
|
TaskLog.syncLogsShutdown(logSyncer);
|
|
TaskLog.syncLogsShutdown(logSyncer);
|
|
}
|
|
}
|
|
|
|
|
|
- private boolean isRecoverySupported(OutputCommitter committer2)
|
|
|
|
- throws IOException {
|
|
|
|
|
|
+ private boolean isRecoverySupported() throws IOException {
|
|
boolean isSupported = false;
|
|
boolean isSupported = false;
|
|
- JobContext _jobContext;
|
|
|
|
|
|
+ Configuration conf = getConfig();
|
|
if (committer != null) {
|
|
if (committer != null) {
|
|
|
|
+ final JobContext _jobContext;
|
|
if (newApiCommitter) {
|
|
if (newApiCommitter) {
|
|
_jobContext = new JobContextImpl(
|
|
_jobContext = new JobContextImpl(
|
|
- getConfig(), TypeConverter.fromYarn(getJobId()));
|
|
|
|
|
|
+ conf, TypeConverter.fromYarn(getJobId()));
|
|
} else {
|
|
} else {
|
|
_jobContext = new org.apache.hadoop.mapred.JobContextImpl(
|
|
_jobContext = new org.apache.hadoop.mapred.JobContextImpl(
|
|
- new JobConf(getConfig()), TypeConverter.fromYarn(getJobId()));
|
|
|
|
|
|
+ new JobConf(conf), TypeConverter.fromYarn(getJobId()));
|
|
}
|
|
}
|
|
- isSupported = committer.isRecoverySupported(_jobContext);
|
|
|
|
|
|
+ isSupported = callWithJobClassLoader(conf,
|
|
|
|
+ new ExceptionAction<Boolean>() {
|
|
|
|
+ public Boolean call(Configuration conf) throws IOException {
|
|
|
|
+ return committer.isRecoverySupported(_jobContext);
|
|
|
|
+ }
|
|
|
|
+ });
|
|
}
|
|
}
|
|
return isSupported;
|
|
return isSupported;
|
|
}
|
|
}
|
|
@@ -1127,7 +1148,7 @@ public class MRAppMaster extends CompositeService {
|
|
MRJobConfig.MR_AM_JOB_RECOVERY_ENABLE,
|
|
MRJobConfig.MR_AM_JOB_RECOVERY_ENABLE,
|
|
MRJobConfig.MR_AM_JOB_RECOVERY_ENABLE_DEFAULT);
|
|
MRJobConfig.MR_AM_JOB_RECOVERY_ENABLE_DEFAULT);
|
|
|
|
|
|
- boolean recoverySupportedByCommitter = isRecoverySupported(committer);
|
|
|
|
|
|
+ boolean recoverySupportedByCommitter = isRecoverySupported();
|
|
|
|
|
|
// If a shuffle secret was not provided by the job client then this app
|
|
// If a shuffle secret was not provided by the job client then this app
|
|
// attempt will generate one. However that disables recovery if there
|
|
// attempt will generate one. However that disables recovery if there
|
|
@@ -1312,7 +1333,7 @@ public class MRAppMaster extends CompositeService {
|
|
this.conf = config;
|
|
this.conf = config;
|
|
}
|
|
}
|
|
@Override
|
|
@Override
|
|
- public void handle(SpeculatorEvent event) {
|
|
|
|
|
|
+ public void handle(final SpeculatorEvent event) {
|
|
if (disabled) {
|
|
if (disabled) {
|
|
return;
|
|
return;
|
|
}
|
|
}
|
|
@@ -1339,7 +1360,12 @@ public class MRAppMaster extends CompositeService {
|
|
if ( (shouldMapSpec && (tType == null || tType == TaskType.MAP))
|
|
if ( (shouldMapSpec && (tType == null || tType == TaskType.MAP))
|
|
|| (shouldReduceSpec && (tType == null || tType == TaskType.REDUCE))) {
|
|
|| (shouldReduceSpec && (tType == null || tType == TaskType.REDUCE))) {
|
|
// Speculator IS enabled, direct the event to there.
|
|
// Speculator IS enabled, direct the event to there.
|
|
- speculator.handle(event);
|
|
|
|
|
|
+ callWithJobClassLoader(conf, new Action<Void>() {
|
|
|
|
+ public Void call(Configuration conf) {
|
|
|
|
+ speculator.handle(event);
|
|
|
|
+ return null;
|
|
|
|
+ }
|
|
|
|
+ });
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
@@ -1499,6 +1525,102 @@ public class MRAppMaster extends CompositeService {
|
|
});
|
|
});
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ /**
|
|
|
|
+ * Creates a job classloader based on the configuration if the job classloader
|
|
|
|
+ * is enabled. It is a no-op if the job classloader is not enabled.
|
|
|
|
+ */
|
|
|
|
+ private void createJobClassLoader(Configuration conf) throws IOException {
|
|
|
|
+ jobClassLoader = MRApps.createJobClassLoader(conf);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Executes the given action with the job classloader set as the configuration
|
|
|
|
+ * classloader as well as the thread context class loader if the job
|
|
|
|
+ * classloader is enabled. After the call, the original classloader is
|
|
|
|
+ * restored.
|
|
|
|
+ *
|
|
|
|
+ * If the job classloader is enabled and the code needs to load user-supplied
|
|
|
|
+ * classes via configuration or thread context classloader, this method should
|
|
|
|
+ * be used in order to load them.
|
|
|
|
+ *
|
|
|
|
+ * @param conf the configuration on which the classloader will be set
|
|
|
|
+ * @param action the callable action to be executed
|
|
|
|
+ */
|
|
|
|
+ <T> T callWithJobClassLoader(Configuration conf, Action<T> action) {
|
|
|
|
+ // if the job classloader is enabled, we may need it to load the (custom)
|
|
|
|
+ // classes; we make the job classloader available and unset it once it is
|
|
|
|
+ // done
|
|
|
|
+ ClassLoader currentClassLoader = conf.getClassLoader();
|
|
|
|
+ boolean setJobClassLoader =
|
|
|
|
+ jobClassLoader != null && currentClassLoader != jobClassLoader;
|
|
|
|
+ if (setJobClassLoader) {
|
|
|
|
+ MRApps.setClassLoader(jobClassLoader, conf);
|
|
|
|
+ }
|
|
|
|
+ try {
|
|
|
|
+ return action.call(conf);
|
|
|
|
+ } finally {
|
|
|
|
+ if (setJobClassLoader) {
|
|
|
|
+ // restore the original classloader
|
|
|
|
+ MRApps.setClassLoader(currentClassLoader, conf);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Executes the given action that can throw a checked exception with the job
|
|
|
|
+ * classloader set as the configuration classloader as well as the thread
|
|
|
|
+ * context class loader if the job classloader is enabled. After the call, the
|
|
|
|
+ * original classloader is restored.
|
|
|
|
+ *
|
|
|
|
+ * If the job classloader is enabled and the code needs to load user-supplied
|
|
|
|
+ * classes via configuration or thread context classloader, this method should
|
|
|
|
+ * be used in order to load them.
|
|
|
|
+ *
|
|
|
|
+ * @param conf the configuration on which the classloader will be set
|
|
|
|
+ * @param action the callable action to be executed
|
|
|
|
+ * @throws IOException if the underlying action throws an IOException
|
|
|
|
+ * @throws YarnRuntimeException if the underlying action throws an exception
|
|
|
|
+ * other than an IOException
|
|
|
|
+ */
|
|
|
|
+ <T> T callWithJobClassLoader(Configuration conf, ExceptionAction<T> action)
|
|
|
|
+ throws IOException {
|
|
|
|
+ // if the job classloader is enabled, we may need it to load the (custom)
|
|
|
|
+ // classes; we make the job classloader available and unset it once it is
|
|
|
|
+ // done
|
|
|
|
+ ClassLoader currentClassLoader = conf.getClassLoader();
|
|
|
|
+ boolean setJobClassLoader =
|
|
|
|
+ jobClassLoader != null && currentClassLoader != jobClassLoader;
|
|
|
|
+ if (setJobClassLoader) {
|
|
|
|
+ MRApps.setClassLoader(jobClassLoader, conf);
|
|
|
|
+ }
|
|
|
|
+ try {
|
|
|
|
+ return action.call(conf);
|
|
|
|
+ } catch (IOException e) {
|
|
|
|
+ throw e;
|
|
|
|
+ } catch (YarnRuntimeException e) {
|
|
|
|
+ throw e;
|
|
|
|
+ } catch (Exception e) {
|
|
|
|
+ // wrap it with a YarnRuntimeException
|
|
|
|
+ throw new YarnRuntimeException(e);
|
|
|
|
+ } finally {
|
|
|
|
+ if (setJobClassLoader) {
|
|
|
|
+ // restore the original classloader
|
|
|
|
+ MRApps.setClassLoader(currentClassLoader, conf);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Action to be wrapped with setting and unsetting the job classloader
|
|
|
|
+ */
|
|
|
|
+ private static interface Action<T> {
|
|
|
|
+ T call(Configuration conf);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private static interface ExceptionAction<T> {
|
|
|
|
+ T call(Configuration conf) throws Exception;
|
|
|
|
+ }
|
|
|
|
+
|
|
@Override
|
|
@Override
|
|
protected void serviceStop() throws Exception {
|
|
protected void serviceStop() throws Exception {
|
|
super.serviceStop();
|
|
super.serviceStop();
|