Browse Source
* fix python task execution error * modify code style * delete AbstractTask#setCommand * migrate blocking task alert feature and common attribute * blocking task migrate(complete base features) * migrate blocking task test * resolve TaskNode.java conflict * fix package problem in BlockingParameters * resolve duplicate code * resolve code style * resolve code style * resolve code style * resolve code style * resolve code style * resolve code style * resolve code style * resolve code style * resolve code style * resolve code style * resolve code style * resolve code style * optimise code based on pr#6283 in ds master repo * add log * modify log * resolve conflicts * change hardcode to variable * Add blocking task UI * refactor old code to compatable with the newest branch * add override annotation * remove old code * remove never used packages * add missing packages declaration * Refine the i18n description * add UT * resolve some code smell problems * add fetch-depth * enable unit test in sonar analysis * disable unit test * remove unused packages * resolve duplicate codes * add logs when task status changed and rename some variables * add BLOCK & READY_BLOCK status and corresponding processing code * revoke UI * revert unit-test Co-authored-by: Jave-Chen <baicai.chen@gmail.com>3.0.0/version-upgrade
Martin Huang
3 years ago
committed by
GitHub
19 changed files with 790 additions and 33 deletions
@ -0,0 +1,35 @@
|
||||
/* |
||||
* Licensed to the Apache Software Foundation (ASF) under one or more |
||||
* contributor license agreements. See the NOTICE file distributed with |
||||
* this work for additional information regarding copyright ownership. |
||||
* The ASF licenses this file to You under the Apache License, Version 2.0 |
||||
* (the "License"); you may not use this file except in compliance with |
||||
* the License. You may obtain a copy of the License at |
||||
* |
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
* |
||||
* Unless required by applicable law or agreed to in writing, software |
||||
* distributed under the License is distributed on an "AS IS" BASIS, |
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. |
||||
* See the License for the specific language governing permissions and |
||||
* limitations under the License. |
||||
*/ |
||||
|
||||
package org.apache.dolphinscheduler.common.enums; |
||||
|
||||
public enum BlockingOpportunity { |
||||
|
||||
BLOCKING_ON_SUCCESS("BlockingOnSuccess"), |
||||
BLOCKING_ON_FAILED("BlockingOnFailed"); |
||||
|
||||
private final String desc; |
||||
|
||||
BlockingOpportunity(String desc){ |
||||
this.desc = desc; |
||||
} |
||||
|
||||
public String getDesc() { |
||||
return desc; |
||||
} |
||||
|
||||
} |
@ -0,0 +1,61 @@
|
||||
/* |
||||
* Licensed to the Apache Software Foundation (ASF) under one or more |
||||
* contributor license agreements. See the NOTICE file distributed with |
||||
* this work for additional information regarding copyright ownership. |
||||
* The ASF licenses this file to You under the Apache License, Version 2.0 |
||||
* (the "License"); you may not use this file except in compliance with |
||||
* the License. You may obtain a copy of the License at |
||||
* |
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
* |
||||
* Unless required by applicable law or agreed to in writing, software |
||||
* distributed under the License is distributed on an "AS IS" BASIS, |
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. |
||||
* See the License for the specific language governing permissions and |
||||
* limitations under the License. |
||||
*/ |
||||
|
||||
package org.apache.dolphinscheduler.common.task.blocking; |
||||
|
||||
import org.apache.dolphinscheduler.common.process.ResourceInfo; |
||||
import org.apache.dolphinscheduler.common.task.AbstractParameters; |
||||
import org.apache.dolphinscheduler.spi.utils.StringUtils; |
||||
|
||||
import java.util.ArrayList; |
||||
import java.util.List; |
||||
|
||||
public class BlockingParameters extends AbstractParameters { |
||||
// condition of blocking: BlockingOnFailed or BlockingOnSuccess
|
||||
|
||||
private String blockingOpportunity; |
||||
|
||||
// if true, alert when blocking, otherwise do nothing
|
||||
|
||||
private boolean isAlertWhenBlocking; |
||||
|
||||
@Override |
||||
public boolean checkParameters() { |
||||
return !StringUtils.isEmpty(blockingOpportunity); |
||||
} |
||||
|
||||
@Override |
||||
public List<ResourceInfo> getResourceFilesList() { |
||||
return new ArrayList<>(); |
||||
} |
||||
|
||||
public String getBlockingOpportunity() { |
||||
return blockingOpportunity; |
||||
} |
||||
|
||||
public void setBlockingCondition(String blockingOpportunity) { |
||||
this.blockingOpportunity = blockingOpportunity; |
||||
} |
||||
|
||||
public boolean isAlertWhenBlocking() { |
||||
return isAlertWhenBlocking; |
||||
} |
||||
|
||||
public void setAlertWhenBlocking(boolean alertWhenBlocking) { |
||||
isAlertWhenBlocking = alertWhenBlocking; |
||||
} |
||||
} |
@ -0,0 +1,197 @@
|
||||
/* |
||||
* Licensed to the Apache Software Foundation (ASF) under one or more |
||||
* contributor license agreements. See the NOTICE file distributed with |
||||
* this work for additional information regarding copyright ownership. |
||||
* The ASF licenses this file to You under the Apache License, Version 2.0 |
||||
* (the "License"); you may not use this file except in compliance with |
||||
* the License. You may obtain a copy of the License at |
||||
* |
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
* |
||||
* Unless required by applicable law or agreed to in writing, software |
||||
* distributed under the License is distributed on an "AS IS" BASIS, |
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. |
||||
* See the License for the specific language governing permissions and |
||||
* limitations under the License. |
||||
*/ |
||||
|
||||
package org.apache.dolphinscheduler.server.master.runner.task; |
||||
|
||||
import org.apache.dolphinscheduler.common.Constants; |
||||
import org.apache.dolphinscheduler.common.enums.BlockingOpportunity; |
||||
import org.apache.dolphinscheduler.common.enums.DependResult; |
||||
import org.apache.dolphinscheduler.common.enums.ExecutionStatus; |
||||
import org.apache.dolphinscheduler.common.enums.TaskType; |
||||
import org.apache.dolphinscheduler.common.model.DependentItem; |
||||
import org.apache.dolphinscheduler.common.model.DependentTaskModel; |
||||
import org.apache.dolphinscheduler.common.task.blocking.BlockingParameters; |
||||
import org.apache.dolphinscheduler.common.task.dependent.DependentParameters; |
||||
import org.apache.dolphinscheduler.common.utils.DependentUtils; |
||||
import org.apache.dolphinscheduler.common.utils.NetUtils; |
||||
import org.apache.dolphinscheduler.common.utils.TaskParametersUtils; |
||||
import org.apache.dolphinscheduler.dao.entity.TaskInstance; |
||||
import org.apache.dolphinscheduler.server.utils.LogUtils; |
||||
|
||||
import java.util.ArrayList; |
||||
import java.util.Date; |
||||
import java.util.List; |
||||
import java.util.Map; |
||||
import java.util.concurrent.ConcurrentHashMap; |
||||
|
||||
import com.google.auto.service.AutoService; |
||||
|
||||
/** |
||||
* blocking task processor |
||||
*/ |
||||
@AutoService(ITaskProcessor.class) |
||||
public class BlockingTaskProcessor extends BaseTaskProcessor { |
||||
|
||||
/** |
||||
* dependent parameters |
||||
*/ |
||||
private DependentParameters dependentParameters; |
||||
|
||||
/** |
||||
* condition result |
||||
*/ |
||||
private DependResult conditionResult = DependResult.WAITING; |
||||
|
||||
/** |
||||
* blocking parameters |
||||
*/ |
||||
private BlockingParameters blockingParam; |
||||
|
||||
/** |
||||
* complete task map |
||||
*/ |
||||
private Map<Long, ExecutionStatus> completeTaskList = new ConcurrentHashMap<>(); |
||||
|
||||
private void initTaskParameters() { |
||||
taskInstance.setLogPath(LogUtils.getTaskLogPath(taskInstance.getFirstSubmitTime(), processInstance.getProcessDefinitionCode(), |
||||
processInstance.getProcessDefinitionVersion(), |
||||
taskInstance.getProcessInstanceId(), |
||||
taskInstance.getId())); |
||||
this.taskInstance.setHost(NetUtils.getAddr(masterConfig.getListenPort())); |
||||
this.taskInstance.setState(ExecutionStatus.RUNNING_EXECUTION); |
||||
this.taskInstance.setStartTime(new Date()); |
||||
this.processService.saveTaskInstance(taskInstance); |
||||
this.dependentParameters = taskInstance.getDependency(); |
||||
this.blockingParam = (BlockingParameters) TaskParametersUtils.getParameters( |
||||
TaskType.BLOCKING.getDesc(), taskInstance.getTaskParams()); |
||||
} |
||||
|
||||
@Override |
||||
protected boolean pauseTask() { |
||||
taskInstance.setState(ExecutionStatus.PAUSE); |
||||
taskInstance.setEndTime(new Date()); |
||||
processService.saveTaskInstance(taskInstance); |
||||
logger.info("blocking task has been paused"); |
||||
return true; |
||||
} |
||||
|
||||
@Override |
||||
protected boolean killTask() { |
||||
taskInstance.setState(ExecutionStatus.KILL); |
||||
taskInstance.setEndTime(new Date()); |
||||
processService.saveTaskInstance(taskInstance); |
||||
logger.info("blocking task has been killed"); |
||||
return true; |
||||
} |
||||
|
||||
@Override |
||||
protected boolean taskTimeout() { |
||||
return true; |
||||
} |
||||
|
||||
@Override |
||||
protected boolean submitTask() { |
||||
this.taskInstance = processService.submitTaskWithRetry(processInstance, taskInstance, maxRetryTimes, commitInterval); |
||||
if (this.taskInstance == null) { |
||||
return false; |
||||
} |
||||
this.setTaskExecutionLogger(); |
||||
initTaskParameters(); |
||||
logger.info("blocking task start"); |
||||
return true; |
||||
} |
||||
|
||||
@Override |
||||
protected boolean runTask() { |
||||
if (conditionResult.equals(DependResult.WAITING)) { |
||||
setConditionResult(); |
||||
endTask(); |
||||
} else { |
||||
endTask(); |
||||
} |
||||
return true; |
||||
} |
||||
|
||||
@Override |
||||
protected boolean dispatchTask() { |
||||
return false; |
||||
} |
||||
|
||||
@Override |
||||
public String getType() { |
||||
return TaskType.BLOCKING.getDesc(); |
||||
} |
||||
|
||||
/** |
||||
* depend result for depend item |
||||
*/ |
||||
private DependResult getDependResultForItem(DependentItem item) { |
||||
|
||||
DependResult dependResult = DependResult.SUCCESS; |
||||
if (!completeTaskList.containsKey(item.getDepTaskCode())) { |
||||
logger.info("depend item: {} have not completed yet.", item.getDepTaskCode()); |
||||
dependResult = DependResult.FAILED; |
||||
return dependResult; |
||||
} |
||||
ExecutionStatus executionStatus = completeTaskList.get(item.getDepTaskCode()); |
||||
if (executionStatus != item.getStatus()) { |
||||
logger.info("depend item : {} expect status: {}, actual status: {}", item.getDepTaskCode(), item.getStatus(), executionStatus); |
||||
dependResult = DependResult.FAILED; |
||||
} |
||||
logger.info("dependent item complete {} {},{}", |
||||
Constants.DEPENDENT_SPLIT, item.getDepTaskCode(), dependResult); |
||||
return dependResult; |
||||
} |
||||
|
||||
private void setConditionResult() { |
||||
|
||||
List<TaskInstance> taskInstances = processService |
||||
.findValidTaskListByProcessId(taskInstance.getProcessInstanceId()); |
||||
for (TaskInstance task : taskInstances) { |
||||
completeTaskList.putIfAbsent(task.getTaskCode(), task.getState()); |
||||
} |
||||
|
||||
List<DependResult> tempResultList = new ArrayList<>(); |
||||
for (DependentTaskModel dependentTaskModel : dependentParameters.getDependTaskList()) { |
||||
List<DependResult> itemDependResult = new ArrayList<>(); |
||||
for (DependentItem item : dependentTaskModel.getDependItemList()) { |
||||
itemDependResult.add(getDependResultForItem(item)); |
||||
} |
||||
DependResult tempResult = DependentUtils.getDependResultForRelation(dependentTaskModel.getRelation(), itemDependResult); |
||||
tempResultList.add(tempResult); |
||||
} |
||||
conditionResult = DependentUtils.getDependResultForRelation(dependentParameters.getRelation(), tempResultList); |
||||
logger.info("the blocking task depend result : {}", conditionResult); |
||||
} |
||||
|
||||
private void endTask() { |
||||
ExecutionStatus status = ExecutionStatus.SUCCESS; |
||||
DependResult expected = this.blockingParam.getBlockingOpportunity() |
||||
.equals(BlockingOpportunity.BLOCKING_ON_SUCCESS.getDesc()) |
||||
? DependResult.SUCCESS : DependResult.FAILED; |
||||
boolean isBlocked = (expected == this.conditionResult); |
||||
logger.info("blocking opportunity: expected-->{}, actual-->{}", expected, this.conditionResult); |
||||
processInstance.setBlocked(isBlocked); |
||||
if (isBlocked) { |
||||
processInstance.setState(ExecutionStatus.READY_BLOCK); |
||||
} |
||||
taskInstance.setState(status); |
||||
taskInstance.setEndTime(new Date()); |
||||
processService.updateTaskInstance(taskInstance); |
||||
logger.info("blocking task execute complete, blocking:{}", isBlocked); |
||||
} |
||||
} |
@ -0,0 +1,270 @@
|
||||
/* |
||||
* Licensed to the Apache Software Foundation (ASF) under one or more |
||||
* contributor license agreements. See the NOTICE file distributed with |
||||
* this work for additional information regarding copyright ownership. |
||||
* The ASF licenses this file to You under the Apache License, Version 2.0 |
||||
* (the "License"); you may not use this file except in compliance with |
||||
* the License. You may obtain a copy of the License at |
||||
* |
||||
* http://www.apache.org/licenses/LICENSE-2.0
|
||||
* |
||||
* Unless required by applicable law or agreed to in writing, software |
||||
* distributed under the License is distributed on an "AS IS" BASIS, |
||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. |
||||
* See the License for the specific language governing permissions and |
||||
* limitations under the License. |
||||
*/ |
||||
|
||||
package org.apache.dolphinscheduler.server.master; |
||||
|
||||
import org.apache.dolphinscheduler.common.enums.DependentRelation; |
||||
import org.apache.dolphinscheduler.common.enums.ExecutionStatus; |
||||
import org.apache.dolphinscheduler.common.enums.TaskTimeoutStrategy; |
||||
import org.apache.dolphinscheduler.common.enums.TaskType; |
||||
import org.apache.dolphinscheduler.common.enums.TimeoutFlag; |
||||
import org.apache.dolphinscheduler.common.model.DependentItem; |
||||
import org.apache.dolphinscheduler.common.model.DependentTaskModel; |
||||
import org.apache.dolphinscheduler.common.model.TaskNode; |
||||
import org.apache.dolphinscheduler.common.task.blocking.BlockingParameters; |
||||
import org.apache.dolphinscheduler.common.task.dependent.DependentParameters; |
||||
import org.apache.dolphinscheduler.common.utils.JSONUtils; |
||||
import org.apache.dolphinscheduler.dao.entity.ProcessInstance; |
||||
import org.apache.dolphinscheduler.dao.entity.TaskDefinition; |
||||
import org.apache.dolphinscheduler.dao.entity.TaskInstance; |
||||
import org.apache.dolphinscheduler.server.master.config.MasterConfig; |
||||
import org.apache.dolphinscheduler.server.master.runner.task.BlockingTaskProcessor; |
||||
import org.apache.dolphinscheduler.server.master.runner.task.TaskAction; |
||||
import org.apache.dolphinscheduler.service.bean.SpringApplicationContext; |
||||
import org.apache.dolphinscheduler.service.process.ProcessService; |
||||
|
||||
import java.util.ArrayList; |
||||
import java.util.Date; |
||||
import java.util.List; |
||||
import java.util.stream.Collectors; |
||||
import java.util.stream.Stream; |
||||
|
||||
import org.junit.Assert; |
||||
import org.junit.Before; |
||||
import org.junit.Test; |
||||
import org.junit.runner.RunWith; |
||||
import org.mockito.Mockito; |
||||
import org.mockito.junit.MockitoJUnitRunner; |
||||
import org.springframework.context.ApplicationContext; |
||||
|
||||
@RunWith(MockitoJUnitRunner.Silent.class) |
||||
public class BlockingTaskTest { |
||||
|
||||
/** |
||||
* TaskNode.runFlag : task can be run normally |
||||
*/ |
||||
public static final String FLOW_NODE_RUN_FLAG_NORMAL = "NORMAL"; |
||||
|
||||
private ProcessService processService; |
||||
|
||||
private ProcessInstance processInstance; |
||||
|
||||
private MasterConfig config; |
||||
|
||||
|
||||
@Before |
||||
public void before() { |
||||
// init spring context
|
||||
ApplicationContext applicationContext = Mockito.mock(ApplicationContext.class); |
||||
SpringApplicationContext springApplicationContext = new SpringApplicationContext(); |
||||
springApplicationContext.setApplicationContext(applicationContext); |
||||
|
||||
// mock master
|
||||
config = new MasterConfig(); |
||||
Mockito.when(applicationContext.getBean(MasterConfig.class)).thenReturn(config); |
||||
config.setTaskCommitRetryTimes(3); |
||||
config.setTaskCommitInterval(1000); |
||||
|
||||
// mock process service
|
||||
processService = Mockito.mock(ProcessService.class); |
||||
Mockito.when(applicationContext.getBean(ProcessService.class)).thenReturn(processService); |
||||
|
||||
// mock process instance
|
||||
processInstance = getProcessInstance(); |
||||
Mockito.when(processService |
||||
.findProcessInstanceById(processInstance.getId())) |
||||
.thenReturn(processInstance); |
||||
|
||||
TaskDefinition taskDefinition = new TaskDefinition(); |
||||
taskDefinition.setTimeoutFlag(TimeoutFlag.OPEN); |
||||
taskDefinition.setTimeoutNotifyStrategy(TaskTimeoutStrategy.WARN); |
||||
taskDefinition.setTimeout(0); |
||||
Mockito.when(processService.findTaskDefinition(1L, 1)) |
||||
.thenReturn(taskDefinition); |
||||
} |
||||
|
||||
private ProcessInstance getProcessInstance() { |
||||
// mock process instance
|
||||
ProcessInstance processInstance = new ProcessInstance(); |
||||
processInstance.setId(1000); |
||||
processInstance.setState(ExecutionStatus.RUNNING_EXECUTION); |
||||
processInstance.setProcessDefinitionCode(1L); |
||||
|
||||
return processInstance; |
||||
} |
||||
|
||||
private TaskInstance getTaskInstance(TaskNode taskNode, ProcessInstance processInstance) { |
||||
// wrap taskNode
|
||||
TaskInstance taskInstance = new TaskInstance(); |
||||
taskInstance.setId(100); |
||||
taskInstance.setName(taskNode.getName()); |
||||
taskInstance.setTaskType(taskNode.getType().toUpperCase()); |
||||
taskInstance.setTaskCode(taskNode.getCode()); |
||||
taskInstance.setTaskDefinitionVersion(taskNode.getVersion()); |
||||
taskInstance.setProcessInstanceId(processInstance.getId()); |
||||
taskInstance.setTaskParams(taskNode.getTaskParams()); |
||||
taskInstance.setState(ExecutionStatus.RUNNING_EXECUTION); |
||||
taskInstance.setFirstSubmitTime(new Date()); |
||||
Mockito.when(processService |
||||
.submitTaskWithRetry(Mockito.any(ProcessInstance.class) |
||||
, Mockito.any(TaskInstance.class) |
||||
, Mockito.any(Integer.class), Mockito.any(Integer.class))) |
||||
.thenReturn(taskInstance); |
||||
return taskInstance; |
||||
} |
||||
|
||||
private TaskNode getTaskNode(String blockingCondition) { |
||||
// mock task nodes
|
||||
// 1----\
|
||||
// 2-----4(Blocking Node)
|
||||
// 3----/
|
||||
// blocking logic: 1-->SUCCESS 2-->SUCCESS 3-->SUCCESS
|
||||
TaskNode taskNode = new TaskNode(); |
||||
taskNode.setId("tasks-1000"); |
||||
taskNode.setName("4"); |
||||
taskNode.setCode(1L); |
||||
taskNode.setVersion(1); |
||||
taskNode.setType(TaskType.BLOCKING.getDesc()); |
||||
taskNode.setRunFlag(FLOW_NODE_RUN_FLAG_NORMAL); |
||||
|
||||
DependentItem dependentItemA = new DependentItem(); |
||||
dependentItemA.setDepTaskCode(1L); |
||||
dependentItemA.setStatus(ExecutionStatus.SUCCESS); |
||||
|
||||
DependentItem dependentItemB = new DependentItem(); |
||||
dependentItemB.setDepTaskCode(2L); |
||||
dependentItemB.setStatus(ExecutionStatus.SUCCESS); |
||||
|
||||
DependentItem dependentItemC = new DependentItem(); |
||||
dependentItemC.setDepTaskCode(3L); |
||||
dependentItemC.setStatus(ExecutionStatus.SUCCESS); |
||||
|
||||
// build relation
|
||||
DependentTaskModel dependentTaskModel = new DependentTaskModel(); |
||||
dependentTaskModel.setDependItemList(Stream.of(dependentItemA, dependentItemB, dependentItemC) |
||||
.collect(Collectors.toList())); |
||||
dependentTaskModel.setRelation(DependentRelation.AND); |
||||
|
||||
DependentParameters dependentParameters = new DependentParameters(); |
||||
dependentParameters.setDependTaskList(Stream.of(dependentTaskModel).collect(Collectors.toList())); |
||||
dependentParameters.setRelation(DependentRelation.AND); |
||||
|
||||
taskNode.setDependence(JSONUtils.toJsonString(dependentParameters)); |
||||
|
||||
// set blocking node params
|
||||
BlockingParameters blockingParameters = new BlockingParameters(); |
||||
blockingParameters.setAlertWhenBlocking(false); |
||||
blockingParameters.setBlockingCondition(blockingCondition); |
||||
|
||||
taskNode.setParams(JSONUtils.toJsonString(blockingParameters)); |
||||
|
||||
return taskNode; |
||||
} |
||||
|
||||
private TaskInstance testBasicInit(String blockingCondition, ExecutionStatus... expectResults) { |
||||
|
||||
TaskInstance taskInstance = getTaskInstance(getTaskNode(blockingCondition), processInstance); |
||||
|
||||
Mockito.when(processService |
||||
.submitTask(processInstance, taskInstance)) |
||||
.thenReturn(taskInstance); |
||||
|
||||
Mockito.when(processService |
||||
.findTaskInstanceById(taskInstance.getId())) |
||||
.thenReturn(taskInstance); |
||||
|
||||
// for BlockingTaskExecThread.initTaskParameters
|
||||
Mockito.when(processService |
||||
.saveTaskInstance(taskInstance)) |
||||
.thenReturn(true); |
||||
|
||||
// for BlockingTaskExecThread.updateTaskState
|
||||
Mockito.when(processService |
||||
.updateTaskInstance(taskInstance)) |
||||
.thenReturn(true); |
||||
|
||||
// for BlockingTaskExecThread.waitTaskQuit
|
||||
List<TaskInstance> conditions = getTaskInstanceForValidTaskList(expectResults); |
||||
Mockito.when(processService. |
||||
findValidTaskListByProcessId(processInstance.getId())) |
||||
.thenReturn(conditions); |
||||
return taskInstance; |
||||
} |
||||
|
||||
/** |
||||
* mock task instance and its execution result in front of blocking node |
||||
*/ |
||||
private List<TaskInstance> getTaskInstanceForValidTaskList(ExecutionStatus... status) { |
||||
List<TaskInstance> taskInstanceList = new ArrayList<>(); |
||||
for (int i = 1; i <= status.length; i++) { |
||||
TaskInstance taskInstance = new TaskInstance(); |
||||
taskInstance.setId(i); |
||||
taskInstance.setName(String.valueOf(i)); |
||||
taskInstance.setState(status[i - 1]); |
||||
taskInstanceList.add(taskInstance); |
||||
} |
||||
return taskInstanceList; |
||||
} |
||||
|
||||
@Test |
||||
public void testBlockingTaskSubmit() { |
||||
TaskInstance taskInstance = testBasicInit("BlockingOnFailed", |
||||
ExecutionStatus.SUCCESS, ExecutionStatus.FAILURE, ExecutionStatus.SUCCESS); |
||||
BlockingTaskProcessor blockingTaskProcessor = new BlockingTaskProcessor(); |
||||
blockingTaskProcessor.init(taskInstance, processInstance); |
||||
boolean res = blockingTaskProcessor.action(TaskAction.SUBMIT); |
||||
Assert.assertEquals(true, res); |
||||
} |
||||
|
||||
@Test |
||||
public void testPauseTask() { |
||||
TaskInstance taskInstance = testBasicInit("BlockingOnFailed", |
||||
ExecutionStatus.SUCCESS, ExecutionStatus.FAILURE, ExecutionStatus.SUCCESS); |
||||
BlockingTaskProcessor blockingTaskProcessor = new BlockingTaskProcessor(); |
||||
blockingTaskProcessor.init(taskInstance, processInstance); |
||||
blockingTaskProcessor.action(TaskAction.SUBMIT); |
||||
blockingTaskProcessor.action(TaskAction.PAUSE); |
||||
ExecutionStatus status = taskInstance.getState(); |
||||
Assert.assertEquals(ExecutionStatus.PAUSE, status); |
||||
} |
||||
|
||||
@Test |
||||
public void testBlocking() { |
||||
TaskInstance taskInstance = testBasicInit("BlockingOnFailed", |
||||
ExecutionStatus.SUCCESS, ExecutionStatus.FAILURE, ExecutionStatus.SUCCESS); |
||||
BlockingTaskProcessor blockingTaskProcessor = new BlockingTaskProcessor(); |
||||
blockingTaskProcessor.init(taskInstance, processInstance); |
||||
blockingTaskProcessor.action(TaskAction.SUBMIT); |
||||
blockingTaskProcessor.action(TaskAction.RUN); |
||||
ExecutionStatus status = processInstance.getState(); |
||||
Assert.assertEquals(ExecutionStatus.READY_BLOCK, status); |
||||
} |
||||
|
||||
@Test |
||||
public void testNoneBlocking() { |
||||
TaskInstance taskInstance = testBasicInit("BlockingOnSuccess", |
||||
ExecutionStatus.SUCCESS, ExecutionStatus.SUCCESS, ExecutionStatus.SUCCESS); |
||||
BlockingTaskProcessor blockingTaskProcessor = new BlockingTaskProcessor(); |
||||
blockingTaskProcessor.init(taskInstance, processInstance); |
||||
blockingTaskProcessor.action(TaskAction.SUBMIT); |
||||
blockingTaskProcessor.action(TaskAction.RUN); |
||||
ExecutionStatus status = processInstance.getState(); |
||||
Assert.assertEquals(ExecutionStatus.RUNNING_EXECUTION, status); |
||||
} |
||||
|
||||
|
||||
} |
Loading…
Reference in new issue