phet commented on code in PR #3893:
URL: https://github.com/apache/gobblin/pull/3893#discussion_r1521697487


##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/DagProcFactory.java:
##########
@@ -35,9 +37,16 @@
 @Alpha
 @Singleton
 public class DagProcFactory implements DagTaskVisitor<DagProc> {
+
+  FlowCompilationValidationHelper flowCompilationValidationHelper;

Review Comment:
   `private final`?



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/DagProcFactory.java:
##########
@@ -35,9 +37,16 @@
 @Alpha
 @Singleton
 public class DagProcFactory implements DagTaskVisitor<DagProc> {
+
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
+
+  @Inject
+  public DagProcFactory(FlowCompilationValidationHelper 
flowCompilationValidationHelper) {
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
+  }

Review Comment:
   missing blank line



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;

Review Comment:
   suggest to place all instance initialization before metric reg, etc.



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId

Review Comment:
   I don't see a `dagId` provided: param is a `Dag<JEP>`
   
   I do however see that the caller already generated (or more accurately, 
deterministically calculated) the ID, and so could have passed it in



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();
+
+     //Submit jobs from the dag ready for execution.
+     for (Dag.DagNode<JobExecutionPlan> dagNode : nextNodes) {
+       submitJob(dagManagementStateStore, dagNode);
+       nextJobNames.add(DagManagerUtils.getJobName(dagNode));
+     }
+
+     log.info("Submitting next nodes for dagId {}, where next jobs to be 
submitted are {}", dagId, nextJobNames);
+
+     //Checkpoint the dag state, it should have an updated value of dag nodes
+     dagManagementStateStore.checkpointDag(dag);
+
+     return nextNodes;
+  }
+
+  /**
+   * Submits a {@link JobSpec} to a {@link SpecExecutor}.
+   */
+  private void submitJob(DagManagementStateStore dagManagementStateStore, 
Dag.DagNode<JobExecutionPlan> dagNode) {
+    DagManagerUtils.incrementJobAttempt(dagNode);
+    JobExecutionPlan jobExecutionPlan = 
DagManagerUtils.getJobExecutionPlan(dagNode);
+    jobExecutionPlan.setExecutionStatus(ExecutionStatus.RUNNING);
+    JobSpec jobSpec = DagManagerUtils.getJobSpec(dagNode);
+    Map<String, String> jobMetadata = 
TimingEventUtils.getJobMetadata(Maps.newHashMap(), jobExecutionPlan);
+
+    String specExecutorUri = DagManagerUtils.getSpecExecutorUri(dagNode);
+
+    // Run this spec on selected executor
+    SpecProducer<Spec> producer;
+    try {
+      dagManagementStateStore.tryAcquireQuota(Collections.singleton(dagNode));
+      producer = DagManagerUtils.getSpecProducer(dagNode);

Review Comment:
   if it's intentional to acquire the quota before even getting the spec 
producer, please note why in a comment



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();
+
+     //Submit jobs from the dag ready for execution.
+     for (Dag.DagNode<JobExecutionPlan> dagNode : nextNodes) {
+       submitJob(dagManagementStateStore, dagNode);

Review Comment:
   each job submission could succeed or fail... do we want to continue on with 
as many succeeding as do?
   
   such a "best effort" dynamic definitely needs documented in a clarifying 
comment.
   
   but the bigger Q is... how do we selectively reattempt submission on some, 
but not others... and what should trigger retry?  in general, gaas must be 
resilient to ephemeral outage in some or even all executors.
   
   relatedly - there's really no limit on how many jobs there could be to 
launch... which we can't predict how long it would take to submit them all.  
that doesn't play well with the time-limited nature of the lease we hold...



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();
+
+     //Submit jobs from the dag ready for execution.
+     for (Dag.DagNode<JobExecutionPlan> dagNode : nextNodes) {
+       submitJob(dagManagementStateStore, dagNode);
+       nextJobNames.add(DagManagerUtils.getJobName(dagNode));
+     }
+
+     log.info("Submitting next nodes for dagId {}, where next jobs to be 
submitted are {}", dagId, nextJobNames);
+
+     //Checkpoint the dag state, it should have an updated value of dag nodes
+     dagManagementStateStore.checkpointDag(dag);
+
+     return nextNodes;
+  }
+
+  /**
+   * Submits a {@link JobSpec} to a {@link SpecExecutor}.
+   */
+  private void submitJob(DagManagementStateStore dagManagementStateStore, 
Dag.DagNode<JobExecutionPlan> dagNode) {
+    DagManagerUtils.incrementJobAttempt(dagNode);
+    JobExecutionPlan jobExecutionPlan = 
DagManagerUtils.getJobExecutionPlan(dagNode);
+    jobExecutionPlan.setExecutionStatus(ExecutionStatus.RUNNING);
+    JobSpec jobSpec = DagManagerUtils.getJobSpec(dagNode);
+    Map<String, String> jobMetadata = 
TimingEventUtils.getJobMetadata(Maps.newHashMap(), jobExecutionPlan);
+
+    String specExecutorUri = DagManagerUtils.getSpecExecutorUri(dagNode);
+
+    // Run this spec on selected executor
+    SpecProducer<Spec> producer;
+    try {
+      dagManagementStateStore.tryAcquireQuota(Collections.singleton(dagNode));
+      producer = DagManagerUtils.getSpecProducer(dagNode);
+      TimingEvent jobOrchestrationTimer = 
eventSubmitter.getTimingEvent(TimingEvent.LauncherTimings.JOB_ORCHESTRATED);
+
+      // Increment job count before submitting the job onto the spec producer, 
in case that throws an exception.
+      // By this point the quota is allocated, so it's imperative to increment 
as missing would introduce the potential to decrement below zero upon quota 
release.
+      // Quota release is guaranteed, despite failure, because exception 
handling within would mark the job FAILED.
+      // When the ensuing kafka message spurs DagManager processing, the quota 
is released and the counts decremented
+      // Ensure that we do not double increment for flows that are retried
+      if (dagNode.getValue().getCurrentAttempts() == 1) {
+        
DagManagementTaskStreamImpl.getDagManagerMetrics().incrementRunningJobMetrics(dagNode);
+      }
+      // Submit the job to the SpecProducer, which in turn performs the actual 
job submission to the SpecExecutor instance.
+      // The SpecProducer implementations submit the job to the underlying 
executor and return when the submission is complete,
+      // either successfully or unsuccessfully. To catch any exceptions in the 
job submission, the DagManagerThread
+      // blocks (by calling Future#get()) until the submission is completed.
+      Future<?> addSpecFuture = producer.addSpec(jobSpec);
+      
dagNode.getValue().setJobFuture(com.google.common.base.Optional.of(addSpecFuture));
+
+      addSpecFuture.get();
+
+      jobMetadata.put(TimingEvent.METADATA_MESSAGE, 
producer.getExecutionLink(addSpecFuture, specExecutorUri));
+      // Add serialized job properties as part of the orchestrated job event 
metadata
+      jobMetadata.put(JobExecutionPlan.JOB_PROPS_KEY, 
dagNode.getValue().toString());
+      jobOrchestrationTimer.stop(jobMetadata);

Review Comment:
   seems like we're doing the work of `sendNotification` ultimately within 
`act`.  let's implement one more `DagProc` and then revisit whether we really 
want to keep `sendNotification`



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;

Review Comment:
   `private final`?



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);

Review Comment:
   next step in refining `DagProc::process` is to get more sophisticated in 
signaling which exceptions are retryable (e.g. `getFlowSpec` timeout) and which 
are hard failures.
   
   consider: if `initialize` raises a retryable exception `process` should 
retry, but when a non-retryable one, such as `URISyntaxException`, then 
`process` itself ought to raise an error.
   
   we must also decide whether `process` would be the one to log the error and 
increment a metric or whether that's for `DagProcEngine`.
   
   any thoughts?



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,

Review Comment:
   name suggests it only works on one node (the 'Next').  maybe 
`submitNextNodes`?



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();

Review Comment:
   IIRC, w/ the `FlowCompilationValidationHelper`, it's not so much that it 
wasn't found, but that compilation failed so no DAG was possible, correct?
   
   if so, this is an error.  next Q then: whose responsibility to increment 
metric, etc.?



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();
+
+     //Submit jobs from the dag ready for execution.
+     for (Dag.DagNode<JobExecutionPlan> dagNode : nextNodes) {
+       submitJob(dagManagementStateStore, dagNode);
+       nextJobNames.add(DagManagerUtils.getJobName(dagNode));
+     }
+
+     log.info("Submitting next nodes for dagId {}, where next jobs to be 
submitted are {}", dagId, nextJobNames);
+
+     //Checkpoint the dag state, it should have an updated value of dag nodes
+     dagManagementStateStore.checkpointDag(dag);
+
+     return nextNodes;
+  }
+
+  /**
+   * Submits a {@link JobSpec} to a {@link SpecExecutor}.
+   */
+  private void submitJob(DagManagementStateStore dagManagementStateStore, 
Dag.DagNode<JobExecutionPlan> dagNode) {
+    DagManagerUtils.incrementJobAttempt(dagNode);
+    JobExecutionPlan jobExecutionPlan = 
DagManagerUtils.getJobExecutionPlan(dagNode);
+    jobExecutionPlan.setExecutionStatus(ExecutionStatus.RUNNING);

Review Comment:
   `RUNNING` already?  it hasn't even been `ORCHESTRATED` on the spec executor. 
 I expected `PENDING`...



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);

Review Comment:
   nit: entire put the message before the ID, as that's easier to machine 
parse/process (e.g. as w/ `grep`)



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();

Review Comment:
   this seems to be used solely for logging.  hence, Q: why not just have one 
log message per job?  in the common case, we have very few simultaneous jobs.  
even in the "worst case" we also have very few.
   
   is there an argument for logging them all together?



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }

Review Comment:
   how did you decide this belongs here rather than inside `submitNext`, which 
is already iterating over these same `Dag.DagNode<>`s?



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();
+
+     //Submit jobs from the dag ready for execution.
+     for (Dag.DagNode<JobExecutionPlan> dagNode : nextNodes) {
+       submitJob(dagManagementStateStore, dagNode);
+       nextJobNames.add(DagManagerUtils.getJobName(dagNode));
+     }
+
+     log.info("Submitting next nodes for dagId {}, where next jobs to be 
submitted are {}", dagId, nextJobNames);
+
+     //Checkpoint the dag state, it should have an updated value of dag nodes
+     dagManagementStateStore.checkpointDag(dag);
+
+     return nextNodes;
+  }
+
+  /**
+   * Submits a {@link JobSpec} to a {@link SpecExecutor}.
+   */
+  private void submitJob(DagManagementStateStore dagManagementStateStore, 
Dag.DagNode<JobExecutionPlan> dagNode) {
+    DagManagerUtils.incrementJobAttempt(dagNode);
+    JobExecutionPlan jobExecutionPlan = 
DagManagerUtils.getJobExecutionPlan(dagNode);
+    jobExecutionPlan.setExecutionStatus(ExecutionStatus.RUNNING);
+    JobSpec jobSpec = DagManagerUtils.getJobSpec(dagNode);
+    Map<String, String> jobMetadata = 
TimingEventUtils.getJobMetadata(Maps.newHashMap(), jobExecutionPlan);
+
+    String specExecutorUri = DagManagerUtils.getSpecExecutorUri(dagNode);
+
+    // Run this spec on selected executor
+    SpecProducer<Spec> producer;
+    try {
+      dagManagementStateStore.tryAcquireQuota(Collections.singleton(dagNode));
+      producer = DagManagerUtils.getSpecProducer(dagNode);
+      TimingEvent jobOrchestrationTimer = 
eventSubmitter.getTimingEvent(TimingEvent.LauncherTimings.JOB_ORCHESTRATED);
+
+      // Increment job count before submitting the job onto the spec producer, 
in case that throws an exception.
+      // By this point the quota is allocated, so it's imperative to increment 
as missing would introduce the potential to decrement below zero upon quota 
release.
+      // Quota release is guaranteed, despite failure, because exception 
handling within would mark the job FAILED.
+      // When the ensuing kafka message spurs DagManager processing, the quota 
is released and the counts decremented
+      // Ensure that we do not double increment for flows that are retried
+      if (dagNode.getValue().getCurrentAttempts() == 1) {
+        
DagManagementTaskStreamImpl.getDagManagerMetrics().incrementRunningJobMetrics(dagNode);

Review Comment:
   reaching back in to grab a `static` suggests we may not yet have the right 
layering.  e.g. should each `DagTask` carry a handle to the DMTSImpl's metrics 
and then provide a `getDagManagerMetrics()` method? 



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();
+
+     //Submit jobs from the dag ready for execution.
+     for (Dag.DagNode<JobExecutionPlan> dagNode : nextNodes) {
+       submitJob(dagManagementStateStore, dagNode);
+       nextJobNames.add(DagManagerUtils.getJobName(dagNode));
+     }
+
+     log.info("Submitting next nodes for dagId {}, where next jobs to be 
submitted are {}", dagId, nextJobNames);
+
+     //Checkpoint the dag state, it should have an updated value of dag nodes
+     dagManagementStateStore.checkpointDag(dag);
+
+     return nextNodes;
+  }
+
+  /**
+   * Submits a {@link JobSpec} to a {@link SpecExecutor}.
+   */
+  private void submitJob(DagManagementStateStore dagManagementStateStore, 
Dag.DagNode<JobExecutionPlan> dagNode) {
+    DagManagerUtils.incrementJobAttempt(dagNode);
+    JobExecutionPlan jobExecutionPlan = 
DagManagerUtils.getJobExecutionPlan(dagNode);
+    jobExecutionPlan.setExecutionStatus(ExecutionStatus.RUNNING);
+    JobSpec jobSpec = DagManagerUtils.getJobSpec(dagNode);
+    Map<String, String> jobMetadata = 
TimingEventUtils.getJobMetadata(Maps.newHashMap(), jobExecutionPlan);
+
+    String specExecutorUri = DagManagerUtils.getSpecExecutorUri(dagNode);
+
+    // Run this spec on selected executor
+    SpecProducer<Spec> producer;
+    try {
+      dagManagementStateStore.tryAcquireQuota(Collections.singleton(dagNode));
+      producer = DagManagerUtils.getSpecProducer(dagNode);
+      TimingEvent jobOrchestrationTimer = 
eventSubmitter.getTimingEvent(TimingEvent.LauncherTimings.JOB_ORCHESTRATED);
+
+      // Increment job count before submitting the job onto the spec producer, 
in case that throws an exception.
+      // By this point the quota is allocated, so it's imperative to increment 
as missing would introduce the potential to decrement below zero upon quota 
release.
+      // Quota release is guaranteed, despite failure, because exception 
handling within would mark the job FAILED.
+      // When the ensuing kafka message spurs DagManager processing, the quota 
is released and the counts decremented
+      // Ensure that we do not double increment for flows that are retried
+      if (dagNode.getValue().getCurrentAttempts() == 1) {

Review Comment:
   nit: for clarify, I'm in favor of always using `DagManagerUtils`':
   ```
   public static JobExecutionPlan getJobExecutionPlan(DagNode<JobExecutionPlan> 
dagNode) {
     return dagNode.getValue();
   }
   ```



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();
+
+     //Submit jobs from the dag ready for execution.
+     for (Dag.DagNode<JobExecutionPlan> dagNode : nextNodes) {
+       submitJob(dagManagementStateStore, dagNode);
+       nextJobNames.add(DagManagerUtils.getJobName(dagNode));
+     }
+
+     log.info("Submitting next nodes for dagId {}, where next jobs to be 
submitted are {}", dagId, nextJobNames);
+
+     //Checkpoint the dag state, it should have an updated value of dag nodes
+     dagManagementStateStore.checkpointDag(dag);
+
+     return nextNodes;
+  }
+
+  /**
+   * Submits a {@link JobSpec} to a {@link SpecExecutor}.
+   */
+  private void submitJob(DagManagementStateStore dagManagementStateStore, 
Dag.DagNode<JobExecutionPlan> dagNode) {
+    DagManagerUtils.incrementJobAttempt(dagNode);
+    JobExecutionPlan jobExecutionPlan = 
DagManagerUtils.getJobExecutionPlan(dagNode);
+    jobExecutionPlan.setExecutionStatus(ExecutionStatus.RUNNING);
+    JobSpec jobSpec = DagManagerUtils.getJobSpec(dagNode);
+    Map<String, String> jobMetadata = 
TimingEventUtils.getJobMetadata(Maps.newHashMap(), jobExecutionPlan);
+
+    String specExecutorUri = DagManagerUtils.getSpecExecutorUri(dagNode);
+
+    // Run this spec on selected executor
+    SpecProducer<Spec> producer;
+    try {
+      dagManagementStateStore.tryAcquireQuota(Collections.singleton(dagNode));
+      producer = DagManagerUtils.getSpecProducer(dagNode);
+      TimingEvent jobOrchestrationTimer = 
eventSubmitter.getTimingEvent(TimingEvent.LauncherTimings.JOB_ORCHESTRATED);
+
+      // Increment job count before submitting the job onto the spec producer, 
in case that throws an exception.
+      // By this point the quota is allocated, so it's imperative to increment 
as missing would introduce the potential to decrement below zero upon quota 
release.
+      // Quota release is guaranteed, despite failure, because exception 
handling within would mark the job FAILED.
+      // When the ensuing kafka message spurs DagManager processing, the quota 
is released and the counts decremented
+      // Ensure that we do not double increment for flows that are retried
+      if (dagNode.getValue().getCurrentAttempts() == 1) {
+        
DagManagementTaskStreamImpl.getDagManagerMetrics().incrementRunningJobMetrics(dagNode);
+      }
+      // Submit the job to the SpecProducer, which in turn performs the actual 
job submission to the SpecExecutor instance.
+      // The SpecProducer implementations submit the job to the underlying 
executor and return when the submission is complete,
+      // either successfully or unsuccessfully. To catch any exceptions in the 
job submission, the DagManagerThread
+      // blocks (by calling Future#get()) until the submission is completed.
+      Future<?> addSpecFuture = producer.addSpec(jobSpec);
+      
dagNode.getValue().setJobFuture(com.google.common.base.Optional.of(addSpecFuture));
+
+      addSpecFuture.get();

Review Comment:
   a. what's the advantage of setting the future in the JEP, if we've already 
forced its result?  we could add the result itself instead..
   
   b. how long could this possibly block?  these `DagProc`s work best when 
blocking is kept to a minimum



##########
gobblin-service/src/main/java/org/apache/gobblin/service/modules/orchestration/proc/LaunchDagProc.java:
##########
@@ -40,36 +65,126 @@
 @Alpha
 public class LaunchDagProc extends DagProc<Optional<Dag<JobExecutionPlan>>, 
Optional<Dag<JobExecutionPlan>>> {
   private final LaunchDagTask launchDagTask;
-  private final AtomicLong orchestrationDelayCounter;
+  FlowCompilationValidationHelper flowCompilationValidationHelper;
 
-  public LaunchDagProc(LaunchDagTask launchDagTask) {
+  public LaunchDagProc(LaunchDagTask launchDagTask, 
FlowCompilationValidationHelper flowCompilationValidationHelper) {
     this.launchDagTask = launchDagTask;
-    this.orchestrationDelayCounter = new AtomicLong(0);
+    AtomicLong orchestrationDelayCounter = new AtomicLong(0);
     ContextAwareGauge<Long> orchestrationDelayMetric = 
metricContext.newContextAwareGauge
         (ServiceMetricNames.FLOW_ORCHESTRATION_DELAY, 
orchestrationDelayCounter::get);
     metricContext.register(orchestrationDelayMetric);
+    this.flowCompilationValidationHelper = flowCompilationValidationHelper;
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> initialize(DagManagementStateStore 
dagManagementStateStore)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    try {
+      DagActionStore.DagAction dagAction = this.launchDagTask.getDagAction();
+      URI flowUri = FlowSpec.Utils.createFlowSpecUri(dagAction.getFlowId());
+      FlowSpec flowSpec = dagManagementStateStore.getFlowSpec(flowUri);
+      flowSpec.addProperty(ConfigurationKeys.FLOW_EXECUTION_ID_KEY, 
dagAction.getFlowExecutionId());
+      return 
this.flowCompilationValidationHelper.createExecutionPlanIfValid(flowSpec).toJavaUtil();
+    } catch (URISyntaxException | SpecNotFoundException | InterruptedException 
e) {
+      throw new RuntimeException(e);
+    }
   }
 
   @Override
   protected Optional<Dag<JobExecutionPlan>> act(DagManagementStateStore 
dagManagementStateStore, Optional<Dag<JobExecutionPlan>> dag)
       throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+    if (!dag.isPresent()) {
+      log.warn("No dag with id " + this.launchDagTask.getDagId() + " found to 
launch");
+      return Optional.empty();
+    }
+    DagManager.DagId dagId = DagManagerUtils.generateDagId(dag.get());
+    Set<Dag.DagNode<JobExecutionPlan>> nextSubmitted = 
submitNext(dagManagementStateStore, dag.get());
+    for (Dag.DagNode<JobExecutionPlan> dagNode : nextSubmitted) {
+      dagManagementStateStore.addDagNodeState(dagNode, dagId);  // compare 
this - arjun1
+    }
+
+    log.info("Dag {} processed.", dagId);
+    return dag;
   }
 
-  @Override
-  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter)
-      throws IOException {
-    throw new UnsupportedOperationException("Not yet implemented");
+  /**
+   * Submit next set of Dag nodes in the Dag identified by the provided dagId
+   */
+   private Set<Dag.DagNode<JobExecutionPlan>> 
submitNext(DagManagementStateStore dagManagementStateStore,
+       Dag<JobExecutionPlan> dag) throws IOException {
+     DagManager.DagId dagId = DagManagerUtils.generateDagId(dag);
+     Set<Dag.DagNode<JobExecutionPlan>> nextNodes = 
DagManagerUtils.getNext(dag);
+     List<String> nextJobNames = new ArrayList<>();
+
+     //Submit jobs from the dag ready for execution.
+     for (Dag.DagNode<JobExecutionPlan> dagNode : nextNodes) {
+       submitJob(dagManagementStateStore, dagNode);
+       nextJobNames.add(DagManagerUtils.getJobName(dagNode));
+     }
+
+     log.info("Submitting next nodes for dagId {}, where next jobs to be 
submitted are {}", dagId, nextJobNames);
+
+     //Checkpoint the dag state, it should have an updated value of dag nodes
+     dagManagementStateStore.checkpointDag(dag);
+
+     return nextNodes;
+  }
+
+  /**
+   * Submits a {@link JobSpec} to a {@link SpecExecutor}.
+   */
+  private void submitJob(DagManagementStateStore dagManagementStateStore, 
Dag.DagNode<JobExecutionPlan> dagNode) {
+    DagManagerUtils.incrementJobAttempt(dagNode);
+    JobExecutionPlan jobExecutionPlan = 
DagManagerUtils.getJobExecutionPlan(dagNode);
+    jobExecutionPlan.setExecutionStatus(ExecutionStatus.RUNNING);
+    JobSpec jobSpec = DagManagerUtils.getJobSpec(dagNode);
+    Map<String, String> jobMetadata = 
TimingEventUtils.getJobMetadata(Maps.newHashMap(), jobExecutionPlan);
+
+    String specExecutorUri = DagManagerUtils.getSpecExecutorUri(dagNode);
+
+    // Run this spec on selected executor
+    SpecProducer<Spec> producer;
+    try {
+      dagManagementStateStore.tryAcquireQuota(Collections.singleton(dagNode));
+      producer = DagManagerUtils.getSpecProducer(dagNode);
+      TimingEvent jobOrchestrationTimer = 
eventSubmitter.getTimingEvent(TimingEvent.LauncherTimings.JOB_ORCHESTRATED);
+
+      // Increment job count before submitting the job onto the spec producer, 
in case that throws an exception.
+      // By this point the quota is allocated, so it's imperative to increment 
as missing would introduce the potential to decrement below zero upon quota 
release.
+      // Quota release is guaranteed, despite failure, because exception 
handling within would mark the job FAILED.
+      // When the ensuing kafka message spurs DagManager processing, the quota 
is released and the counts decremented
+      // Ensure that we do not double increment for flows that are retried
+      if (dagNode.getValue().getCurrentAttempts() == 1) {
+        
DagManagementTaskStreamImpl.getDagManagerMetrics().incrementRunningJobMetrics(dagNode);
+      }
+      // Submit the job to the SpecProducer, which in turn performs the actual 
job submission to the SpecExecutor instance.
+      // The SpecProducer implementations submit the job to the underlying 
executor and return when the submission is complete,
+      // either successfully or unsuccessfully. To catch any exceptions in the 
job submission, the DagManagerThread
+      // blocks (by calling Future#get()) until the submission is completed.
+      Future<?> addSpecFuture = producer.addSpec(jobSpec);
+      
dagNode.getValue().setJobFuture(com.google.common.base.Optional.of(addSpecFuture));
+
+      addSpecFuture.get();
+
+      jobMetadata.put(TimingEvent.METADATA_MESSAGE, 
producer.getExecutionLink(addSpecFuture, specExecutorUri));
+      // Add serialized job properties as part of the orchestrated job event 
metadata
+      jobMetadata.put(JobExecutionPlan.JOB_PROPS_KEY, 
dagNode.getValue().toString());
+      jobOrchestrationTimer.stop(jobMetadata);
+      log.info("Orchestrated job: {} on Executor: {}", 
DagManagerUtils.getFullyQualifiedJobName(dagNode), specExecutorUri);
+      
DagManagementTaskStreamImpl.getDagManagerMetrics().incrementJobsSentToExecutor(dagNode);
+    } catch (Exception e) {
+      TimingEvent jobFailedTimer = 
eventSubmitter.getTimingEvent(TimingEvent.LauncherTimings.JOB_FAILED);
+      String message = "Cannot submit job " + 
DagManagerUtils.getFullyQualifiedJobName(dagNode) + " on executor " + 
specExecutorUri;
+      log.error(message, e);
+      jobMetadata.put(TimingEvent.METADATA_MESSAGE, message + " due to " + 
e.getMessage());
+      if (jobFailedTimer != null) {
+        jobFailedTimer.stop(jobMetadata);
+      }
+    }
   }
 
   @Override
-  protected void commit(DagManagementStateStore dagManagementStateStore, 
Optional<Dag<JobExecutionPlan>> dag) {
+  protected void sendNotification(Optional<Dag<JobExecutionPlan>> result, 
EventSubmitter eventSubmitter) {
     throw new UnsupportedOperationException("Not yet implemented");

Review Comment:
   we don't really want to throw this, so better would be empty impl.
   
   (BTW, a `LaunchDagProcTest` almost certainly would have caught this)



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]

Reply via email to