@ -14,12 +14,13 @@
* See the License for the specific language governing permissions and
* limitations under the License .
* /
package org.apache.dolphinscheduler.server.master.runner ;
import static org.apache.dolphinscheduler.common.Constants.CMDPARAM_COMPLEMENT_DATA_END_DATE ;
import static org.apache.dolphinscheduler.common.Constants.CMDPARAM_COMPLEMENT_DATA_START_DATE ;
import static org.apache.dolphinscheduler.common.Constants.CMDPARAM_RECOVERY_START_NODE_STRING ;
import static org.apache.dolphinscheduler.common.Constants.CMDPARAM_START_NODE_NAMES ;
import static org.apache.dolphinscheduler.common.Constants.CMD_ PARAM_RECOVERY_START_NODE_STRING ;
import static org.apache.dolphinscheduler.common.Constants.CMD_ PARAM_START_NODE_NAMES ;
import static org.apache.dolphinscheduler.common.Constants.DEFAULT_WORKER_GROUP ;
import static org.apache.dolphinscheduler.common.Constants.SEC_2_MINUTES_TIME_UNIT ;
@ -35,7 +36,6 @@ import org.apache.dolphinscheduler.common.graph.DAG;
import org.apache.dolphinscheduler.common.model.TaskNode ;
import org.apache.dolphinscheduler.common.model.TaskNodeRelation ;
import org.apache.dolphinscheduler.common.process.ProcessDag ;
import org.apache.dolphinscheduler.common.task.conditions.ConditionsParameters ;
import org.apache.dolphinscheduler.common.thread.Stopper ;
import org.apache.dolphinscheduler.common.thread.ThreadUtils ;
import org.apache.dolphinscheduler.common.utils.CollectionUtils ;
@ -87,22 +87,18 @@ public class MasterExecThread implements Runnable {
* logger of MasterExecThread
* /
private static final Logger logger = LoggerFactory . getLogger ( MasterExecThread . class ) ;
/ * *
* process instance
* /
private ProcessInstance processInstance ;
/ * *
* runing TaskNode
* /
private final Map < MasterBaseTaskExecThread , Future < Boolean > > activeTaskNode = new ConcurrentHashMap < > ( ) ;
/ * *
* task exec service
* /
private final ExecutorService taskExecService ;
/ * *
* process instance
* /
private ProcessInstance processInstance ;
/ * *
* submit failure nodes
* /
@ -172,9 +168,16 @@ public class MasterExecThread implements Runnable {
*
* /
private NettyRemotingClient nettyRemotingClient ;
/ * *
* submit post node
*
* @param parentNodeName parent node name
* /
private Map < String , Object > propToValue = new ConcurrentHashMap < String , Object > ( ) ;
/ * *
* constructor of MasterExecThread
*
* @param processInstance processInstance
* @param processService processService
* @param nettyRemotingClient nettyRemotingClient
@ -195,9 +198,6 @@ public class MasterExecThread implements Runnable {
this . alertManager = alertManager ;
}
@Override
public void run ( ) {
@ -236,6 +236,7 @@ public class MasterExecThread implements Runnable {
/ * *
* execute process
*
* @throws Exception exception
* /
private void executeProcess ( ) throws Exception {
@ -246,6 +247,7 @@ public class MasterExecThread implements Runnable {
/ * *
* execute complement process
*
* @throws Exception exception
* /
private void executeComplementProcess ( ) throws Exception {
@ -326,8 +328,8 @@ public class MasterExecThread implements Runnable {
// flow end
// execute next process instance complement data
processInstance . setScheduleTime ( scheduleDate ) ;
if ( cmdParam . containsKey ( Constants . CMDPARAM_RECOVERY_START_NODE_STRING ) ) {
cmdParam . remove ( Constants . CMDPARAM_RECOVERY_START_NODE_STRING ) ;
if ( cmdParam . containsKey ( Constants . CMD_ PARAM_RECOVERY_START_NODE_STRING ) ) {
cmdParam . remove ( Constants . CMD_ PARAM_RECOVERY_START_NODE_STRING ) ;
processInstance . setCommandParam ( JSONUtils . toJsonString ( cmdParam ) ) ;
}
@ -343,9 +345,9 @@ public class MasterExecThread implements Runnable {
}
}
/ * *
* prepare process parameter
*
* @throws Exception exception
* /
private void prepareProcess ( ) throws Exception {
@ -358,7 +360,6 @@ public class MasterExecThread implements Runnable {
logger . info ( "prepare process :{} end" , processInstance . getId ( ) ) ;
}
/ * *
* process end handle
* /
@ -372,9 +373,9 @@ public class MasterExecThread implements Runnable {
alertManager . sendAlertProcessInstance ( processInstance , taskInstances ) ;
}
/ * *
* generate process dag
*
* @throws Exception exception
* /
private void buildFlowDag ( ) throws Exception {
@ -441,6 +442,7 @@ public class MasterExecThread implements Runnable {
/ * *
* submit task to execute
*
* @param taskInstance task instance
* @return TaskInstance
* /
@ -463,6 +465,7 @@ public class MasterExecThread implements Runnable {
/ * *
* find task instance in db .
* in case submit more than one same name task in the same time .
*
* @param taskName task name
* @return TaskInstance
* /
@ -478,6 +481,7 @@ public class MasterExecThread implements Runnable {
/ * *
* encapsulation task
*
* @param processInstance process instance
* @param nodeName node name
* @return TaskInstance
@ -540,11 +544,6 @@ public class MasterExecThread implements Runnable {
return taskInstance ;
}
/ * *
* submit post node
* @param parentNodeName parent node name
* /
private Map < String , Object > propToValue = new ConcurrentHashMap < String , Object > ( ) ;
private void submitPostNode ( String parentNodeName ) {
Set < String > submitTaskNodeList = DagHelper . parsePostNodes ( parentNodeName , skipTaskNodeList , dag , completeTaskList ) ;
List < TaskInstance > taskInstances = new ArrayList < > ( ) ;
@ -582,6 +581,7 @@ public class MasterExecThread implements Runnable {
/ * *
* determine whether the dependencies of the task node are complete
*
* @return DependResult
* /
private DependResult isTaskDepsComplete ( String taskName ) {
@ -621,9 +621,6 @@ public class MasterExecThread implements Runnable {
/ * *
* depend node is completed , but here need check the condition task branch is the next node
* @param dependNodeName
* @param nextNodeName
* @return
* /
private boolean dependTaskSuccess ( String dependNodeName , String nextNodeName ) {
if ( dag . getNode ( dependNodeName ) . isConditionsTask ( ) ) {
@ -643,6 +640,7 @@ public class MasterExecThread implements Runnable {
/ * *
* query task instance by complete state
*
* @param state state
* @return task instance list
* /
@ -658,6 +656,7 @@ public class MasterExecThread implements Runnable {
/ * *
* where there are ongoing tasks
*
* @param state state
* @return ExecutionStatus
* /
@ -708,6 +707,7 @@ public class MasterExecThread implements Runnable {
/ * *
* whether task for waiting thread
*
* @return Boolean whether has waiting thread task
* /
private boolean hasWaitingThreadTask ( ) {
@ -720,6 +720,7 @@ public class MasterExecThread implements Runnable {
* 1 , failed retry task in the preparation queue , returns to failure directly
* 2 , exists pause task , complement not completed , pending submission of tasks , return to suspension
* 3 , success
*
* @return ExecutionStatus
* /
private ExecutionStatus processReadyPause ( ) {
@ -737,9 +738,9 @@ public class MasterExecThread implements Runnable {
}
}
/ * *
* generate the latest process instance status by the tasks state
*
* @return process instance execution status
* /
private ExecutionStatus getProcessInstanceState ( ) {
@ -798,7 +799,6 @@ public class MasterExecThread implements Runnable {
/ * *
* whether standby task list have retry tasks
* @return
* /
private boolean retryTaskExists ( ) {
@ -816,6 +816,7 @@ public class MasterExecThread implements Runnable {
/ * *
* whether complement end
*
* @return Boolean whether is complement end
* /
private boolean isComplementEnd ( ) {
@ -856,6 +857,7 @@ public class MasterExecThread implements Runnable {
/ * *
* get task dependency result
*
* @param taskInstance task instance
* @return DependResult
* /
@ -865,6 +867,7 @@ public class MasterExecThread implements Runnable {
/ * *
* add task to standby list
*
* @param taskInstance task instance
* /
private void addTaskToStandByList ( TaskInstance taskInstance ) {
@ -874,6 +877,7 @@ public class MasterExecThread implements Runnable {
/ * *
* remove task from stand by list
*
* @param taskInstance task instance
* /
private void removeTaskFromStandbyList ( TaskInstance taskInstance ) {
@ -883,6 +887,7 @@ public class MasterExecThread implements Runnable {
/ * *
* has retry task in standby
*
* @return Boolean whether has retry task in standby
* /
private boolean hasRetryTaskInStandBy ( ) {
@ -1000,6 +1005,7 @@ public class MasterExecThread implements Runnable {
/ * *
* whether check process time out
*
* @param processInstance task instance
* @return true if time out of process instance > running time of process instance
* /
@ -1016,13 +1022,13 @@ public class MasterExecThread implements Runnable {
/ * *
* whether can submit task to queue
*
* @return boolean
* /
private boolean canSubmitTaskToQueue ( ) {
return OSUtils . checkResource ( masterConfig . getMasterMaxCpuloadAvg ( ) , masterConfig . getMasterReservedMemory ( ) ) ;
}
/ * *
* close the on going tasks
* /
@ -1052,6 +1058,7 @@ public class MasterExecThread implements Runnable {
/ * *
* whether the retry interval is timed out
*
* @param taskInstance task instance
* @return Boolean
* /
@ -1059,8 +1066,10 @@ public class MasterExecThread implements Runnable {
if ( taskInstance . getState ( ) ! = ExecutionStatus . FAILURE ) {
return true ;
}
if ( taskInstance . getId ( ) = = 0 | |
taskInstance . getMaxRetryTimes ( ) = = 0 | |
if ( taskInstance . getId ( ) = = 0
| |
taskInstance . getMaxRetryTimes ( ) = = 0
| |
taskInstance . getRetryInterval ( ) = = 0 ) {
return true ;
}
@ -1093,6 +1102,7 @@ public class MasterExecThread implements Runnable {
/ * *
* get recovery task instance
*
* @param taskId task id
* @return recovery task instance
* /
@ -1116,6 +1126,7 @@ public class MasterExecThread implements Runnable {
/ * *
* get start task instance list
*
* @param cmdParam command param
* @return task instance list
* /
@ -1124,8 +1135,8 @@ public class MasterExecThread implements Runnable {
List < TaskInstance > instanceList = new ArrayList < > ( ) ;
Map < String , String > paramMap = JSONUtils . toMap ( cmdParam ) ;
if ( paramMap ! = null & & paramMap . containsKey ( CMDPARAM_RECOVERY_START_NODE_STRING ) ) {
String [ ] idList = paramMap . get ( CMDPARAM_RECOVERY_START_NODE_STRING ) . split ( Constants . COMMA ) ;
if ( paramMap ! = null & & paramMap . containsKey ( CMD_ PARAM_RECOVERY_START_NODE_STRING ) ) {
String [ ] idList = paramMap . get ( CMD_ PARAM_RECOVERY_START_NODE_STRING ) . split ( Constants . COMMA ) ;
for ( String nodeId : idList ) {
TaskInstance task = getRecoveryTaskInstance ( nodeId ) ;
if ( task ! = null ) {
@ -1138,6 +1149,7 @@ public class MasterExecThread implements Runnable {
/ * *
* parse "StartNodeNameList" from cmd param
*
* @param cmdParam command param
* @return start node name list
* /
@ -1147,8 +1159,8 @@ public class MasterExecThread implements Runnable {
if ( paramMap = = null ) {
return startNodeNameList ;
}
if ( paramMap . containsKey ( CMDPARAM_START_NODE_NAMES ) ) {
startNodeNameList = Arrays . asList ( paramMap . get ( CMDPARAM_START_NODE_NAMES ) . split ( Constants . COMMA ) ) ;
if ( paramMap . containsKey ( CMD_ PARAM_START_NODE_NAMES ) ) {
startNodeNameList = Arrays . asList ( paramMap . get ( CMD_ PARAM_START_NODE_NAMES ) . split ( Constants . COMMA ) ) ;
}
return startNodeNameList ;
}
@ -1156,6 +1168,7 @@ public class MasterExecThread implements Runnable {
/ * *
* generate start node name list from parsing command param ;
* if "StartNodeIdList" exists in command param , return StartNodeIdList
*
* @return recovery node name list
* /
private List < String > getRecoveryNodeNameList ( ) {
@ -1170,6 +1183,7 @@ public class MasterExecThread implements Runnable {
/ * *
* generate flow dag
*
* @param processDefinitionJson process definition json
* @param startNodeNameList start node name list
* @param recoveryNodeNameList recovery node name list