From cb8cac3cc746e5e4f24b47ec1c02042a7426e4e8 Mon Sep 17 00:00:00 2001 From: Aravindan Ramkumar <1028385+aravindanr@users.noreply.github.com> Date: Fri, 9 Apr 2021 09:49:48 -0700 Subject: [PATCH] removed Task NAME constants and used constants from TaskType --- .../common/metadata/tasks/TaskDef.java | 70 ++++++++-------- .../contribs/tasks/http/HttpTask.java | 10 +-- .../tasks/kafka/KafkaPublishTask.java | 29 +++---- .../sqs/DefaultEventQueueProcessorTest.java | 28 +++---- .../queue/DefaultEventQueueProcessor.java | 34 ++++---- .../core/execution/WorkflowExecutor.java | 7 +- .../execution/mapper/EventTaskMapper.java | 25 +++--- .../execution/mapper/LambdaTaskMapper.java | 25 +++--- .../mapper/SubWorkflowTaskMapper.java | 25 +++--- .../execution/mapper/TerminateTaskMapper.java | 24 +++--- .../core/execution/mapper/WaitTaskMapper.java | 24 +++--- .../core/execution/tasks/Decision.java | 30 ++++--- .../core/execution/tasks/DoWhile.java | 29 ++++--- .../conductor/core/execution/tasks/Event.java | 27 +++--- .../core/execution/tasks/ExclusiveJoin.java | 30 +++---- .../conductor/core/execution/tasks/Fork.java | 31 ++++--- .../conductor/core/execution/tasks/Join.java | 30 ++++--- .../core/execution/tasks/Lambda.java | 28 +++---- .../core/execution/tasks/SetVariable.java | 30 ++++--- .../core/execution/tasks/SubWorkflow.java | 26 +++--- .../core/execution/tasks/Terminate.java | 37 ++++----- .../conductor/core/execution/tasks/Wait.java | 30 ++++--- .../execution/tasks/WorkflowSystemTask.java | 2 +- .../conductor/core/utils/ParametersUtils.java | 3 +- .../WorkflowTaskTypeConstraint.java | 26 +++--- .../core/execution/TestDeciderOutcomes.java | 5 +- .../core/execution/TestDeciderService.java | 6 +- .../core/execution/TestWorkflowExecutor.java | 11 ++- .../execution/TestWorkflowRepairService.java | 12 +-- .../mapper/SubWorkflowTaskMapperTest.java | 26 +++--- .../execution/mapper/WaitTaskMapperTest.java | 24 +++--- .../TestSystemTaskWorkerCoordinator.java | 32 ++++---- ...hicalForkJoinSubworkflowRestartSpec.groovy | 28 +++---- ...rchicalForkJoinSubworkflowRetrySpec.groovy | 54 ++++++------ .../NestedForkJoinSubWorkflowSpec.groovy | 82 +++++++++---------- 35 files changed, 467 insertions(+), 473 deletions(-) diff --git a/common/src/main/java/com/netflix/conductor/common/metadata/tasks/TaskDef.java b/common/src/main/java/com/netflix/conductor/common/metadata/tasks/TaskDef.java index 7cf456caf3..29b57e3b3c 100644 --- a/common/src/main/java/com/netflix/conductor/common/metadata/tasks/TaskDef.java +++ b/common/src/main/java/com/netflix/conductor/common/metadata/tasks/TaskDef.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.common.metadata.tasks; @@ -90,24 +90,24 @@ public enum RetryLogic {FIXED, EXPONENTIAL_BACKOFF} // @ProtoField(id = 13) // private Integer rateLimitPerSecond; - @ProtoField(id = 14) + @ProtoField(id = 14) private Integer rateLimitPerFrequency; - @ProtoField(id = 15) + @ProtoField(id = 15) private Integer rateLimitFrequencyInSeconds; - @ProtoField(id = 16) + @ProtoField(id = 16) private String isolationGroupId; - @ProtoField(id = 17) + @ProtoField(id = 17) private String executionNameSpace; - @ProtoField(id = 18) + @ProtoField(id = 18) @OwnerEmailMandatoryConstraint @Email(message = "ownerEmail should be valid email address") private String ownerEmail; - @ProtoField(id = 19) + @ProtoField(id = 19) @Min(value = 0, message = "TaskDef pollTimeoutSeconds: {value} must be >= 0") private Integer pollTimeoutSeconds; @@ -131,7 +131,7 @@ public TaskDef(String name, String description, int retryCount, long timeoutSeco } public TaskDef(String name, String description, String ownerEmail, int retryCount, - long timeoutSeconds, long responseTimeoutSeconds) { + long timeoutSeconds, long responseTimeoutSeconds) { this.name = name; this.description = description; this.ownerEmail = ownerEmail; @@ -411,29 +411,29 @@ public boolean equals(Object o) { } TaskDef taskDef = (TaskDef) o; return getRetryCount() == taskDef.getRetryCount() && - getTimeoutSeconds() == taskDef.getTimeoutSeconds() && - getRetryDelaySeconds() == taskDef.getRetryDelaySeconds() && - getResponseTimeoutSeconds() == taskDef.getResponseTimeoutSeconds() && - Objects.equals(getName(), taskDef.getName()) && - Objects.equals(getDescription(), taskDef.getDescription()) && - Objects.equals(getInputKeys(), taskDef.getInputKeys()) && - Objects.equals(getOutputKeys(), taskDef.getOutputKeys()) && - getTimeoutPolicy() == taskDef.getTimeoutPolicy() && - getRetryLogic() == taskDef.getRetryLogic() && - Objects.equals(getConcurrentExecLimit(), taskDef.getConcurrentExecLimit()) && - Objects.equals(getRateLimitPerFrequency(), taskDef.getRateLimitPerFrequency()) && - Objects.equals(getInputTemplate(), taskDef.getInputTemplate()) && - Objects.equals(getIsolationGroupId(), taskDef.getIsolationGroupId()) && - Objects.equals(getExecutionNameSpace(), taskDef.getExecutionNameSpace()) && - Objects.equals(getOwnerEmail(), taskDef.getOwnerEmail()); + getTimeoutSeconds() == taskDef.getTimeoutSeconds() && + getRetryDelaySeconds() == taskDef.getRetryDelaySeconds() && + getResponseTimeoutSeconds() == taskDef.getResponseTimeoutSeconds() && + Objects.equals(getName(), taskDef.getName()) && + Objects.equals(getDescription(), taskDef.getDescription()) && + Objects.equals(getInputKeys(), taskDef.getInputKeys()) && + Objects.equals(getOutputKeys(), taskDef.getOutputKeys()) && + getTimeoutPolicy() == taskDef.getTimeoutPolicy() && + getRetryLogic() == taskDef.getRetryLogic() && + Objects.equals(getConcurrentExecLimit(), taskDef.getConcurrentExecLimit()) && + Objects.equals(getRateLimitPerFrequency(), taskDef.getRateLimitPerFrequency()) && + Objects.equals(getInputTemplate(), taskDef.getInputTemplate()) && + Objects.equals(getIsolationGroupId(), taskDef.getIsolationGroupId()) && + Objects.equals(getExecutionNameSpace(), taskDef.getExecutionNameSpace()) && + Objects.equals(getOwnerEmail(), taskDef.getOwnerEmail()); } @Override public int hashCode() { return Objects.hash(getName(), getDescription(), getRetryCount(), getTimeoutSeconds(), getInputKeys(), - getOutputKeys(), getTimeoutPolicy(), getRetryLogic(), getRetryDelaySeconds(), - getResponseTimeoutSeconds(), getConcurrentExecLimit(), getRateLimitPerFrequency(), getInputTemplate(), - getIsolationGroupId(), getExecutionNameSpace(), getOwnerEmail()); + getOutputKeys(), getTimeoutPolicy(), getRetryLogic(), getRetryDelaySeconds(), + getResponseTimeoutSeconds(), getConcurrentExecLimit(), getRateLimitPerFrequency(), getInputTemplate(), + getIsolationGroupId(), getExecutionNameSpace(), getOwnerEmail()); } } diff --git a/contribs/src/main/java/com/netflix/conductor/contribs/tasks/http/HttpTask.java b/contribs/src/main/java/com/netflix/conductor/contribs/tasks/http/HttpTask.java index 018162c122..db4c967cd0 100644 --- a/contribs/src/main/java/com/netflix/conductor/contribs/tasks/http/HttpTask.java +++ b/contribs/src/main/java/com/netflix/conductor/contribs/tasks/http/HttpTask.java @@ -23,7 +23,6 @@ import com.netflix.conductor.core.utils.Utils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.http.HttpEntity; import org.springframework.http.HttpHeaders; import org.springframework.http.HttpMethod; @@ -40,10 +39,12 @@ import java.util.List; import java.util.Map; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_HTTP; + /** * Task that enables calling another HTTP endpoint as part of its execution */ -@Component(HttpTask.NAME) +@Component(TASK_TYPE_HTTP) public class HttpTask extends WorkflowSystemTask { private static final Logger LOGGER = LoggerFactory.getLogger(HttpTask.class); @@ -53,8 +54,6 @@ public class HttpTask extends WorkflowSystemTask { static final String MISSING_REQUEST = "Missing HTTP request. Task input MUST have a '" + REQUEST_PARAMETER_NAME + "' key with HttpTask.Input as value. See documentation for HttpTask for required input parameters"; - public static final String NAME = "HTTP"; - private final TypeReference> mapOfObj = new TypeReference>() { }; private final TypeReference> listOfObj = new TypeReference>() { @@ -63,10 +62,9 @@ public class HttpTask extends WorkflowSystemTask { protected RestTemplateProvider restTemplateProvider; private final String requestParameter; - @Autowired public HttpTask(RestTemplateProvider restTemplateProvider, ObjectMapper objectMapper) { - this(NAME, restTemplateProvider, objectMapper); + this(TASK_TYPE_HTTP, restTemplateProvider, objectMapper); } public HttpTask(String name, diff --git a/contribs/src/main/java/com/netflix/conductor/contribs/tasks/kafka/KafkaPublishTask.java b/contribs/src/main/java/com/netflix/conductor/contribs/tasks/kafka/KafkaPublishTask.java index 7da6ed9b1c..db1213166b 100644 --- a/contribs/src/main/java/com/netflix/conductor/contribs/tasks/kafka/KafkaPublishTask.java +++ b/contribs/src/main/java/com/netflix/conductor/contribs/tasks/kafka/KafkaPublishTask.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.contribs.tasks.kafka; @@ -41,13 +41,14 @@ import java.util.concurrent.Future; import java.util.stream.Collectors; -@Component(KafkaPublishTask.NAME) +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_KAFKA_PUBLISH; + +@Component(TASK_TYPE_KAFKA_PUBLISH) public class KafkaPublishTask extends WorkflowSystemTask { private static final Logger LOGGER = LoggerFactory.getLogger(KafkaPublishTask.class); static final String REQUEST_PARAMETER_NAME = "kafka_request"; - public static final String NAME = "KAFKA_PUBLISH"; private static final String MISSING_REQUEST = "Missing Kafka request. Task input MUST have a '" + REQUEST_PARAMETER_NAME + "' key with KafkaTask.Input as value. See documentation for KafkaTask for required input parameters"; @@ -62,11 +63,11 @@ public class KafkaPublishTask extends WorkflowSystemTask { @Autowired public KafkaPublishTask(KafkaProducerManager clientManager, ObjectMapper objectMapper) { - super(NAME); + super(TASK_TYPE_KAFKA_PUBLISH); this.requestParameter = REQUEST_PARAMETER_NAME; this.producerManager = clientManager; this.objectMapper = objectMapper; - LOGGER.info("KafkaTask initialized..."); + LOGGER.info("KafkaTask initialized."); } @Override diff --git a/contribs/src/test/java/com/netflix/conductor/contribs/queue/sqs/DefaultEventQueueProcessorTest.java b/contribs/src/test/java/com/netflix/conductor/contribs/queue/sqs/DefaultEventQueueProcessorTest.java index d65658cd57..22891a9703 100644 --- a/contribs/src/test/java/com/netflix/conductor/contribs/queue/sqs/DefaultEventQueueProcessorTest.java +++ b/contribs/src/test/java/com/netflix/conductor/contribs/queue/sqs/DefaultEventQueueProcessorTest.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.contribs.queue.sqs; @@ -18,10 +18,9 @@ import com.netflix.conductor.common.metadata.tasks.Task; import com.netflix.conductor.common.metadata.tasks.Task.Status; import com.netflix.conductor.common.run.Workflow; +import com.netflix.conductor.core.events.queue.DefaultEventQueueProcessor; import com.netflix.conductor.core.events.queue.Message; import com.netflix.conductor.core.events.queue.ObservableQueue; -import com.netflix.conductor.core.events.queue.DefaultEventQueueProcessor; -import com.netflix.conductor.core.execution.tasks.Wait; import com.netflix.conductor.service.ExecutionService; import org.junit.Before; import org.junit.BeforeClass; @@ -38,6 +37,7 @@ import java.util.Map; import java.util.concurrent.TimeUnit; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_WAIT; import static org.junit.Assert.assertNotNull; import static org.junit.Assert.assertTrue; import static org.mockito.ArgumentMatchers.any; @@ -91,7 +91,7 @@ public static void setup() { task0.setStatus(Status.IN_PROGRESS); task0.setTaskId("t0"); task0.setReferenceTaskName("t0"); - task0.setTaskType(Wait.NAME); + task0.setTaskType(TASK_TYPE_WAIT); Workflow workflow0 = new Workflow(); workflow0.setWorkflowId("v_0"); workflow0.getTasks().add(task0); @@ -99,7 +99,7 @@ public static void setup() { Task task2 = new Task(); task2.setStatus(Status.IN_PROGRESS); task2.setTaskId("t2"); - task2.setTaskType(Wait.NAME); + task2.setTaskType(TASK_TYPE_WAIT); Workflow workflow2 = new Workflow(); workflow2.setWorkflowId("v_2"); workflow2.getTasks().add(task2); diff --git a/core/src/main/java/com/netflix/conductor/core/events/queue/DefaultEventQueueProcessor.java b/core/src/main/java/com/netflix/conductor/core/events/queue/DefaultEventQueueProcessor.java index a647c72a63..7b60fba4ff 100644 --- a/core/src/main/java/com/netflix/conductor/core/events/queue/DefaultEventQueueProcessor.java +++ b/core/src/main/java/com/netflix/conductor/core/events/queue/DefaultEventQueueProcessor.java @@ -1,14 +1,14 @@ /* - * Copyright 2021 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.events.queue; @@ -21,18 +21,20 @@ import com.netflix.conductor.common.run.Workflow; import com.netflix.conductor.core.exception.ApplicationException; import com.netflix.conductor.core.exception.ApplicationException.Code; -import com.netflix.conductor.core.execution.tasks.Wait; import com.netflix.conductor.service.ExecutionService; +import org.apache.commons.lang3.StringUtils; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; +import org.springframework.stereotype.Component; + import java.util.Collections; import java.util.HashMap; import java.util.List; import java.util.Map; import java.util.Optional; import java.util.UUID; -import org.apache.commons.lang3.StringUtils; -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; -import org.springframework.stereotype.Component; + +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_WAIT; /** * Monitors and processes messages on the default event queues that Conductor listens on. @@ -93,7 +95,7 @@ private void startMonitor(Status status, ObservableQueue queue) { payload); taskOptional = workflow.getTasks().stream() .filter(task -> !task.getStatus().isTerminal() && task.getTaskType().equals( - Wait.NAME)).findFirst(); + TASK_TYPE_WAIT)).findFirst(); } else { taskOptional = workflow.getTasks().stream().filter( task -> !task.getStatus().isTerminal() && task.getReferenceTaskName().equals(taskRefName)) diff --git a/core/src/main/java/com/netflix/conductor/core/execution/WorkflowExecutor.java b/core/src/main/java/com/netflix/conductor/core/execution/WorkflowExecutor.java index 887730d496..48399ce86d 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/WorkflowExecutor.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/WorkflowExecutor.java @@ -35,7 +35,6 @@ import com.netflix.conductor.core.exception.ApplicationException; import com.netflix.conductor.core.exception.ApplicationException.Code; import com.netflix.conductor.core.exception.TerminateWorkflowException; -import com.netflix.conductor.core.execution.tasks.SubWorkflow; import com.netflix.conductor.core.execution.tasks.SystemTaskRegistry; import com.netflix.conductor.core.execution.tasks.Terminate; import com.netflix.conductor.core.execution.tasks.WorkflowSystemTask; @@ -1717,7 +1716,7 @@ private boolean rerunWF(String workflowId, String taskId, Map ta // If not found look into sub workflows if (rerunFromTask == null) { for (Task task : workflow.getTasks()) { - if (task.getTaskType().equalsIgnoreCase(SubWorkflow.NAME)) { + if (task.getTaskType().equalsIgnoreCase(TASK_TYPE_SUB_WORKFLOW)) { String subWorkflowId = task.getSubWorkflowId(); if (rerunWF(subWorkflowId, taskId, taskInput, null, null)) { rerunFromTask = task; @@ -1760,7 +1759,7 @@ private boolean rerunWF(String workflowId, String taskId, Map ta rerunFromTask.setRetried(false); rerunFromTask.setExecuted(false); rerunFromTask.setExternalOutputPayloadStoragePath(null); - if (rerunFromTask.getTaskType().equalsIgnoreCase(SubWorkflow.NAME)) { + if (rerunFromTask.getTaskType().equalsIgnoreCase(TASK_TYPE_SUB_WORKFLOW)) { // if task is sub workflow set task as IN_PROGRESS and reset start time rerunFromTask.setStatus(IN_PROGRESS); rerunFromTask.setStartTime(System.currentTimeMillis()); @@ -1812,7 +1811,7 @@ void updateParentWorkflowTask(Workflow subWorkflow) { } private void executeSubworkflowTaskAndSyncData(Workflow subWorkflow, Task subWorkflowTask) { - WorkflowSystemTask subWorkflowSystemTask = systemTaskRegistry.get(SubWorkflow.NAME); + WorkflowSystemTask subWorkflowSystemTask = systemTaskRegistry.get(TASK_TYPE_SUB_WORKFLOW); subWorkflowSystemTask.execute(subWorkflow, subWorkflowTask, this); // Keep Subworkflow task's data consistent with Subworkflow's. if (subWorkflowTask.getStatus().isTerminal() && subWorkflowTask.getExternalOutputPayloadStoragePath() != null diff --git a/core/src/main/java/com/netflix/conductor/core/execution/mapper/EventTaskMapper.java b/core/src/main/java/com/netflix/conductor/core/execution/mapper/EventTaskMapper.java index 4b97143c11..ad8b777cec 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/mapper/EventTaskMapper.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/mapper/EventTaskMapper.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.mapper; @@ -16,7 +16,6 @@ import com.netflix.conductor.common.metadata.tasks.TaskType; import com.netflix.conductor.common.metadata.workflow.WorkflowTask; import com.netflix.conductor.common.run.Workflow; -import com.netflix.conductor.core.execution.tasks.Event; import com.netflix.conductor.core.utils.ParametersUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -27,6 +26,8 @@ import java.util.List; import java.util.Map; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_EVENT; + @Component public class EventTaskMapper implements TaskMapper { @@ -61,7 +62,7 @@ public List getMappedTasks(TaskMapperContext taskMapperContext) { Boolean asynComplete = (Boolean) eventTaskInput.get("asyncComplete"); Task eventTask = new Task(); - eventTask.setTaskType(Event.NAME); + eventTask.setTaskType(TASK_TYPE_EVENT); eventTask.setTaskDefName(taskToSchedule.getName()); eventTask.setReferenceTaskName(taskToSchedule.getTaskReferenceName()); eventTask.setWorkflowInstanceId(workflowInstance.getWorkflowId()); diff --git a/core/src/main/java/com/netflix/conductor/core/execution/mapper/LambdaTaskMapper.java b/core/src/main/java/com/netflix/conductor/core/execution/mapper/LambdaTaskMapper.java index cde93938c1..f0bde2809a 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/mapper/LambdaTaskMapper.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/mapper/LambdaTaskMapper.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.mapper; @@ -17,7 +17,6 @@ import com.netflix.conductor.common.metadata.tasks.TaskType; import com.netflix.conductor.common.metadata.workflow.WorkflowTask; import com.netflix.conductor.common.run.Workflow; -import com.netflix.conductor.core.execution.tasks.Lambda; import com.netflix.conductor.core.utils.ParametersUtils; import com.netflix.conductor.dao.MetadataDAO; import org.slf4j.Logger; @@ -29,6 +28,8 @@ import java.util.Map; import java.util.Optional; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_LAMBDA; + /** * @author x-ultra */ @@ -67,7 +68,7 @@ public List getMappedTasks(TaskMapperContext taskMapperContext) { taskDefinition); Task lambdaTask = new Task(); - lambdaTask.setTaskType(Lambda.NAME); + lambdaTask.setTaskType(TASK_TYPE_LAMBDA); lambdaTask.setTaskDefName(taskMapperContext.getTaskToSchedule().getName()); lambdaTask.setReferenceTaskName(taskMapperContext.getTaskToSchedule().getTaskReferenceName()); lambdaTask.setWorkflowInstanceId(workflowInstance.getWorkflowId()); diff --git a/core/src/main/java/com/netflix/conductor/core/execution/mapper/SubWorkflowTaskMapper.java b/core/src/main/java/com/netflix/conductor/core/execution/mapper/SubWorkflowTaskMapper.java index f20f201472..5a549ca7ee 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/mapper/SubWorkflowTaskMapper.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/mapper/SubWorkflowTaskMapper.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.mapper; @@ -20,7 +20,6 @@ import com.netflix.conductor.common.metadata.workflow.WorkflowTask; import com.netflix.conductor.common.run.Workflow; import com.netflix.conductor.core.exception.TerminateWorkflowException; -import com.netflix.conductor.core.execution.tasks.SubWorkflow; import com.netflix.conductor.core.utils.ParametersUtils; import com.netflix.conductor.dao.MetadataDAO; import org.slf4j.Logger; @@ -33,6 +32,8 @@ import java.util.Map; import java.util.Optional; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW; + @Component public class SubWorkflowTaskMapper implements TaskMapper { @@ -75,7 +76,7 @@ public List getMappedTasks(TaskMapperContext taskMapperContext) { } Task subWorkflowTask = new Task(); - subWorkflowTask.setTaskType(SubWorkflow.NAME); + subWorkflowTask.setTaskType(TASK_TYPE_SUB_WORKFLOW); subWorkflowTask.setTaskDefName(taskToSchedule.getName()); subWorkflowTask.setReferenceTaskName(taskToSchedule.getTaskReferenceName()); subWorkflowTask.setWorkflowInstanceId(workflowInstance.getWorkflowId()); diff --git a/core/src/main/java/com/netflix/conductor/core/execution/mapper/TerminateTaskMapper.java b/core/src/main/java/com/netflix/conductor/core/execution/mapper/TerminateTaskMapper.java index 4d9f1e25fa..d32166a0b2 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/mapper/TerminateTaskMapper.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/mapper/TerminateTaskMapper.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.mapper; @@ -16,7 +16,6 @@ import com.netflix.conductor.common.metadata.tasks.TaskType; import com.netflix.conductor.common.metadata.workflow.WorkflowTask; import com.netflix.conductor.common.run.Workflow; -import com.netflix.conductor.core.execution.tasks.Terminate; import com.netflix.conductor.core.utils.ParametersUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -25,6 +24,7 @@ import java.util.List; import java.util.Map; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_TERMINATE; import static java.util.Collections.singletonList; @Component @@ -55,7 +55,7 @@ public List getMappedTasks(TaskMapperContext taskMapperContext) { .getTaskInputV2(taskMapperContext.getTaskToSchedule().getInputParameters(), workflowInstance, taskId, null); Task task = new Task(); - task.setTaskType(Terminate.NAME); + task.setTaskType(TASK_TYPE_TERMINATE); task.setTaskDefName(taskToSchedule.getName()); task.setReferenceTaskName(taskToSchedule.getTaskReferenceName()); task.setWorkflowInstanceId(workflowInstance.getWorkflowId()); diff --git a/core/src/main/java/com/netflix/conductor/core/execution/mapper/WaitTaskMapper.java b/core/src/main/java/com/netflix/conductor/core/execution/mapper/WaitTaskMapper.java index 4d31f76426..f2536f1cee 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/mapper/WaitTaskMapper.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/mapper/WaitTaskMapper.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.mapper; @@ -26,6 +26,8 @@ import java.util.List; import java.util.Map; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_WAIT; + /** * An implementation of {@link TaskMapper} to map a {@link WorkflowTask} of type {@link TaskType#WAIT} to a {@link Task} * of type {@link Wait} with {@link Task.Status#IN_PROGRESS} @@ -60,7 +62,7 @@ public List getMappedTasks(TaskMapperContext taskMapperContext) { workflowInstance, taskId, null); Task waitTask = new Task(); - waitTask.setTaskType(Wait.NAME); + waitTask.setTaskType(TASK_TYPE_WAIT); waitTask.setTaskDefName(taskMapperContext.getTaskToSchedule().getName()); waitTask.setReferenceTaskName(taskMapperContext.getTaskToSchedule().getTaskReferenceName()); waitTask.setWorkflowInstanceId(workflowInstance.getWorkflowId()); diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Decision.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Decision.java index 43029355d7..44cb352e14 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Decision.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Decision.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -16,17 +16,15 @@ import com.netflix.conductor.common.metadata.tasks.Task.Status; import com.netflix.conductor.common.run.Workflow; import com.netflix.conductor.core.execution.WorkflowExecutor; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; -@Component(Decision.NAME) -public class Decision extends WorkflowSystemTask { +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_DECISION; - public static final String NAME = "DECISION"; +@Component(TASK_TYPE_DECISION) +public class Decision extends WorkflowSystemTask { - @Autowired public Decision() { - super(NAME); + super(TASK_TYPE_DECISION); } @Override diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/DoWhile.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/DoWhile.java index 2f2fbaf89a..e757c65342 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/DoWhile.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/DoWhile.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -24,7 +24,6 @@ import com.netflix.conductor.core.utils.ParametersUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; import javax.script.ScriptException; @@ -35,17 +34,17 @@ import java.util.Map; import java.util.stream.Collectors; -@Component(DoWhile.NAME) +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_DO_WHILE; + +@Component(TASK_TYPE_DO_WHILE) public class DoWhile extends WorkflowSystemTask { private static final Logger LOGGER = LoggerFactory.getLogger(DoWhile.class); - public static final String NAME = "DO_WHILE"; private final ParametersUtils parametersUtils; - @Autowired public DoWhile(ParametersUtils parametersUtils) { - super(NAME); + super(TASK_TYPE_DO_WHILE); this.parametersUtils = parametersUtils; } diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Event.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Event.java index 2fab44f2d2..f3a8e7b418 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Event.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Event.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -26,16 +26,16 @@ import com.netflix.conductor.core.utils.ParametersUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; import java.util.Collections; import java.util.HashMap; import java.util.Map; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_EVENT; import static com.netflix.conductor.core.exception.ApplicationException.Code.INTERNAL_ERROR; -@Component(Event.NAME) +@Component(TASK_TYPE_EVENT) public class Event extends WorkflowSystemTask { private static final Logger LOGGER = LoggerFactory.getLogger(Event.class); @@ -45,9 +45,8 @@ public class Event extends WorkflowSystemTask { private final ParametersUtils parametersUtils; private final EventQueues eventQueues; - @Autowired public Event(EventQueues eventQueues, ParametersUtils parametersUtils, ObjectMapper objectMapper) { - super(NAME); + super(TASK_TYPE_EVENT); this.parametersUtils = parametersUtils; this.eventQueues = eventQueues; this.objectMapper = objectMapper; diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/ExclusiveJoin.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/ExclusiveJoin.java index 7e7d88670e..a54e32ffba 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/ExclusiveJoin.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/ExclusiveJoin.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -18,22 +18,22 @@ import com.netflix.conductor.core.execution.WorkflowExecutor; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; import java.util.List; import java.util.stream.Collectors; -@Component(ExclusiveJoin.NAME) +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_EXCLUSIVE_JOIN; + +@Component(TASK_TYPE_EXCLUSIVE_JOIN) public class ExclusiveJoin extends WorkflowSystemTask { private static final Logger LOGGER = LoggerFactory.getLogger(ExclusiveJoin.class); - public static final String NAME = "EXCLUSIVE_JOIN"; + private static final String DEFAULT_EXCLUSIVE_JOIN_TASKS = "defaultExclusiveJoinTask"; - @Autowired public ExclusiveJoin() { - super(NAME); + super(TASK_TYPE_EXCLUSIVE_JOIN); } @Override diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Fork.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Fork.java index 36bda46ebc..6011ee5e6d 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Fork.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Fork.java @@ -1,27 +1,26 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; -@Component(Fork.NAME) -public class Fork extends WorkflowSystemTask { +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_FORK; - public static final String NAME = "FORK"; +@Component(TASK_TYPE_FORK) +public class Fork extends WorkflowSystemTask { - @Autowired public Fork() { - super(NAME); + super(TASK_TYPE_FORK); } + } diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Join.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Join.java index 9721d37961..bbaa7bdf20 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Join.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Join.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -17,20 +17,18 @@ import com.netflix.conductor.common.run.Workflow; import com.netflix.conductor.common.utils.TaskUtils; import com.netflix.conductor.core.execution.WorkflowExecutor; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; import java.util.List; import java.util.stream.Collectors; -@Component(Join.NAME) -public class Join extends WorkflowSystemTask { +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_JOIN; - public static final String NAME = "JOIN"; +@Component(TASK_TYPE_JOIN) +public class Join extends WorkflowSystemTask { - @Autowired public Join() { - super(NAME); + super(TASK_TYPE_JOIN); } @Override diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Lambda.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Lambda.java index dc84387168..1d71a47a0e 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Lambda.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Lambda.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -19,11 +19,12 @@ import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; import java.util.Map; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_LAMBDA; + /** * @author X-Ultra *

@@ -47,16 +48,15 @@ * * then to use task output, e.g. script_test.output.testvalue */ -@Component(Lambda.NAME) +@Component(TASK_TYPE_LAMBDA) public class Lambda extends WorkflowSystemTask { private static final Logger LOGGER = LoggerFactory.getLogger(Lambda.class); private static final String QUERY_EXPRESSION_PARAMETER = "scriptExpression"; public static final String NAME = "LAMBDA"; - @Autowired public Lambda() { - super(NAME); + super(TASK_TYPE_LAMBDA); } @Override diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/SetVariable.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/SetVariable.java index 2a63b69761..38846b8d29 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/SetVariable.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/SetVariable.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -20,7 +20,6 @@ import com.netflix.conductor.core.execution.WorkflowExecutor; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; import java.io.ByteArrayOutputStream; @@ -29,21 +28,20 @@ import java.util.HashMap; import java.util.Map; -@Component(SetVariable.NAME) +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SET_VARIABLE; + +@Component(TASK_TYPE_SET_VARIABLE) public class SetVariable extends WorkflowSystemTask { private static final Logger LOGGER = LoggerFactory.getLogger(SetVariable.class); - public static final String NAME = "SET_VARIABLE"; private final ConductorProperties properties; private final ObjectMapper objectMapper; - @Autowired public SetVariable(ConductorProperties properties, ObjectMapper objectMapper) { - super(NAME); + super(TASK_TYPE_SET_VARIABLE); this.properties = properties; this.objectMapper = objectMapper; - LOGGER.info(NAME + " task initialized..."); } private boolean validateVariablesSize(Workflow workflow, Task task, Map variables) { diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/SubWorkflow.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/SubWorkflow.java index 49fe0fa448..c9161dfee1 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/SubWorkflow.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/SubWorkflow.java @@ -1,14 +1,14 @@ /* - * Copyright 2021 Netflix, Inc. + * Copyright 2021 Netflix, Inc. *

* Licensed 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. + * 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 com.netflix.conductor.core.execution.tasks; @@ -23,24 +23,22 @@ import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; import java.util.Map; -@Component(SubWorkflow.NAME) +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW; + +@Component(TASK_TYPE_SUB_WORKFLOW) public class SubWorkflow extends WorkflowSystemTask { private static final Logger LOGGER = LoggerFactory.getLogger(SubWorkflow.class); private static final String SUB_WORKFLOW_ID = "subWorkflowId"; - public static final String NAME = "SUB_WORKFLOW"; - private final ObjectMapper objectMapper; - @Autowired public SubWorkflow(ObjectMapper objectMapper) { - super(NAME); + super(TASK_TYPE_SUB_WORKFLOW); this.objectMapper = objectMapper; } diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Terminate.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Terminate.java index c2b450dd1c..32aea66418 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Terminate.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Terminate.java @@ -1,27 +1,28 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; -import static com.netflix.conductor.common.run.Workflow.WorkflowStatus.COMPLETED; -import static com.netflix.conductor.common.run.Workflow.WorkflowStatus.FAILED; - import com.netflix.conductor.common.metadata.tasks.Task; import com.netflix.conductor.common.run.Workflow; import com.netflix.conductor.core.execution.WorkflowExecutor; +import org.springframework.stereotype.Component; + import java.util.HashMap; import java.util.Map; -import org.springframework.beans.factory.annotation.Autowired; -import org.springframework.stereotype.Component; + +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_TERMINATE; +import static com.netflix.conductor.common.run.Workflow.WorkflowStatus.COMPLETED; +import static com.netflix.conductor.common.run.Workflow.WorkflowStatus.FAILED; /** * Task that can terminate a workflow with a given status and modify the workflow's output with a given parameter, it @@ -50,16 +51,14 @@ * This task has some validations on creation and execution, they are: - the "terminationStatus" parameter is mandatory * and it can only receive the values "COMPLETED" or "FAILED" - the terminate task cannot be optional */ -@Component(Terminate.NAME) +@Component(TASK_TYPE_TERMINATE) public class Terminate extends WorkflowSystemTask { private static final String TERMINATION_STATUS_PARAMETER = "terminationStatus"; private static final String TERMINATION_WORKFLOW_OUTPUT = "workflowOutput"; - public static final String NAME = "TERMINATE"; - @Autowired public Terminate() { - super(NAME); + super(TASK_TYPE_TERMINATE); } @Override diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Wait.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Wait.java index c85527717f..cdd5989dc9 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/Wait.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/Wait.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -16,17 +16,15 @@ import com.netflix.conductor.common.metadata.tasks.Task.Status; import com.netflix.conductor.common.run.Workflow; import com.netflix.conductor.core.execution.WorkflowExecutor; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; -@Component(Wait.NAME) -public class Wait extends WorkflowSystemTask { +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_WAIT; - public static final String NAME = "WAIT"; +@Component(TASK_TYPE_WAIT) +public class Wait extends WorkflowSystemTask { - @Autowired public Wait() { - super(NAME); + super(TASK_TYPE_WAIT); } @Override diff --git a/core/src/main/java/com/netflix/conductor/core/execution/tasks/WorkflowSystemTask.java b/core/src/main/java/com/netflix/conductor/core/execution/tasks/WorkflowSystemTask.java index 22b55197f9..ee7531dc00 100644 --- a/core/src/main/java/com/netflix/conductor/core/execution/tasks/WorkflowSystemTask.java +++ b/core/src/main/java/com/netflix/conductor/core/execution/tasks/WorkflowSystemTask.java @@ -19,7 +19,7 @@ import java.util.Optional; -public class WorkflowSystemTask { +public abstract class WorkflowSystemTask { private final String name; diff --git a/core/src/main/java/com/netflix/conductor/core/utils/ParametersUtils.java b/core/src/main/java/com/netflix/conductor/core/utils/ParametersUtils.java index 8aa0eef1ce..d65076d0c5 100644 --- a/core/src/main/java/com/netflix/conductor/core/utils/ParametersUtils.java +++ b/core/src/main/java/com/netflix/conductor/core/utils/ParametersUtils.java @@ -45,8 +45,7 @@ public class ParametersUtils { private static final Logger LOGGER = LoggerFactory.getLogger(ParametersUtils.class); private final ObjectMapper objectMapper; - private final TypeReference> map = new TypeReference>() { - }; + private final TypeReference> map = new TypeReference>() {}; public ParametersUtils(ObjectMapper objectMapper) { this.objectMapper = objectMapper; diff --git a/core/src/main/java/com/netflix/conductor/validations/WorkflowTaskTypeConstraint.java b/core/src/main/java/com/netflix/conductor/validations/WorkflowTaskTypeConstraint.java index 0f00de84ff..2727250a58 100644 --- a/core/src/main/java/com/netflix/conductor/validations/WorkflowTaskTypeConstraint.java +++ b/core/src/main/java/com/netflix/conductor/validations/WorkflowTaskTypeConstraint.java @@ -1,21 +1,20 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.validations; import com.netflix.conductor.common.metadata.tasks.TaskDef; import com.netflix.conductor.common.metadata.tasks.TaskType; import com.netflix.conductor.common.metadata.workflow.WorkflowTask; -import com.netflix.conductor.core.execution.tasks.SubWorkflow; import javax.validation.Constraint; import javax.validation.ConstraintValidator; @@ -27,6 +26,7 @@ import java.lang.annotation.Target; import java.util.Optional; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW; import static com.netflix.conductor.core.execution.tasks.Terminate.getTerminationStatusParameter; import static com.netflix.conductor.core.execution.tasks.Terminate.validateInputStatus; import static java.lang.annotation.ElementType.ANNOTATION_TYPE; @@ -90,7 +90,7 @@ public boolean isValid(WorkflowTask workflowTask, ConstraintValidatorContext con case TaskType.TASK_TYPE_DO_WHILE: valid = isDoWhileTaskValid(workflowTask, context); break; - case TaskType.TASK_TYPE_SUB_WORKFLOW: + case TASK_TYPE_SUB_WORKFLOW: valid = isSubWorkflowTaskValid(workflowTask, context); break; case TaskType.TASK_TYPE_JSON_JQ_TRANSFORM: @@ -151,7 +151,7 @@ private boolean isDoWhileTaskValid(WorkflowTask workflowTask, ConstraintValidato context.buildConstraintViolationWithTemplate(message).addConstraintViolation(); valid = false; } - if (workflowTask.collectTasks().stream().anyMatch(t -> t.getType().equals(SubWorkflow.NAME))) { + if (workflowTask.collectTasks().stream().anyMatch(t -> t.getType().equals(TASK_TYPE_SUB_WORKFLOW))) { String message = String .format("SUB_WORKFLOW task inside loopover task: %s is not supported.", workflowTask.getName()); context.buildConstraintViolationWithTemplate(message).addConstraintViolation(); diff --git a/core/src/test/java/com/netflix/conductor/core/execution/TestDeciderOutcomes.java b/core/src/test/java/com/netflix/conductor/core/execution/TestDeciderOutcomes.java index 78c1ab56de..0df9d68e40 100644 --- a/core/src/test/java/com/netflix/conductor/core/execution/TestDeciderOutcomes.java +++ b/core/src/test/java/com/netflix/conductor/core/execution/TestDeciderOutcomes.java @@ -71,6 +71,7 @@ import static com.netflix.conductor.common.metadata.tasks.TaskType.JOIN; import static com.netflix.conductor.common.metadata.tasks.TaskType.SIMPLE; import static com.netflix.conductor.common.metadata.tasks.TaskType.SUB_WORKFLOW; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_DECISION; import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_FORK; import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_JOIN; import static com.netflix.conductor.common.metadata.tasks.TaskType.USER_DEFINED; @@ -99,12 +100,12 @@ public class TestDeciderOutcomes { @Configuration public static class TestConfiguration { - @Bean(Decision.NAME) + @Bean(TASK_TYPE_DECISION) public Decision decision() { return new Decision(); } - @Bean(Join.NAME) + @Bean(TASK_TYPE_JOIN) public Join join() { return new Join(); } diff --git a/core/src/test/java/com/netflix/conductor/core/execution/TestDeciderService.java b/core/src/test/java/com/netflix/conductor/core/execution/TestDeciderService.java index 94f8b37b20..fbfb717546 100644 --- a/core/src/test/java/com/netflix/conductor/core/execution/TestDeciderService.java +++ b/core/src/test/java/com/netflix/conductor/core/execution/TestDeciderService.java @@ -40,7 +40,6 @@ import com.netflix.conductor.core.execution.mapper.UserDefinedTaskMapper; import com.netflix.conductor.core.execution.mapper.WaitTaskMapper; import com.netflix.conductor.core.execution.tasks.SystemTaskRegistry; -import com.netflix.conductor.core.execution.tasks.Terminate; import com.netflix.conductor.core.utils.ExternalPayloadStorageUtils; import com.netflix.conductor.core.utils.ParametersUtils; import com.netflix.conductor.dao.MetadataDAO; @@ -83,6 +82,7 @@ import static com.netflix.conductor.common.metadata.tasks.TaskType.JOIN; import static com.netflix.conductor.common.metadata.tasks.TaskType.SIMPLE; import static com.netflix.conductor.common.metadata.tasks.TaskType.SUB_WORKFLOW; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_TERMINATE; import static com.netflix.conductor.common.metadata.tasks.TaskType.USER_DEFINED; import static com.netflix.conductor.common.metadata.tasks.TaskType.WAIT; import static org.junit.Assert.assertEquals; @@ -1053,9 +1053,9 @@ public void testUpdateWorkflowOutput_WhenDefinitionHasOutputParameters() { public void testUpdateWorkflowOutput_WhenWorkflowHasTerminateTask() { Workflow workflow = new Workflow(); Task task = new Task(); - task.setTaskType(Terminate.NAME); + task.setTaskType(TASK_TYPE_TERMINATE); task.setStatus(Status.COMPLETED); - task.setOutputData(new HashMap() {{ + task.setOutputData(new HashMap() {{ put("taskKey", "taskValue"); }}); workflow.getTasks().add(task); diff --git a/core/src/test/java/com/netflix/conductor/core/execution/TestWorkflowExecutor.java b/core/src/test/java/com/netflix/conductor/core/execution/TestWorkflowExecutor.java index 4a20057653..0e955bf9ba 100644 --- a/core/src/test/java/com/netflix/conductor/core/execution/TestWorkflowExecutor.java +++ b/core/src/test/java/com/netflix/conductor/core/execution/TestWorkflowExecutor.java @@ -93,6 +93,9 @@ import static com.netflix.conductor.common.metadata.tasks.TaskType.LAMBDA; import static com.netflix.conductor.common.metadata.tasks.TaskType.SIMPLE; import static com.netflix.conductor.common.metadata.tasks.TaskType.SUB_WORKFLOW; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_LAMBDA; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_WAIT; import static com.netflix.conductor.common.metadata.tasks.TaskType.USER_DEFINED; import static com.netflix.conductor.common.metadata.tasks.TaskType.WAIT; import static java.util.Comparator.comparingInt; @@ -131,17 +134,17 @@ public class TestWorkflowExecutor { @Configuration public static class TestConfiguration { - @Bean(SubWorkflow.NAME) + @Bean(TASK_TYPE_SUB_WORKFLOW) public SubWorkflow subWorkflow(ObjectMapper objectMapper) { return new SubWorkflow(objectMapper); } - @Bean(Lambda.NAME) + @Bean(TASK_TYPE_LAMBDA) public Lambda lambda() { return new Lambda(); } - @Bean(Wait.NAME) + @Bean(TASK_TYPE_WAIT) public Wait waitBean() { return new Wait(); } @@ -254,7 +257,7 @@ public void testScheduleTask() { task1.setWorkflowTask(taskToSchedule); Task task2 = new Task(); - task2.setTaskType(Wait.NAME); + task2.setTaskType(TASK_TYPE_WAIT); task2.setTaskDefName(taskToSchedule.getName()); task2.setReferenceTaskName(taskToSchedule.getTaskReferenceName()); task2.setWorkflowInstanceId(workflow.getWorkflowId()); diff --git a/core/src/test/java/com/netflix/conductor/core/execution/TestWorkflowRepairService.java b/core/src/test/java/com/netflix/conductor/core/execution/TestWorkflowRepairService.java index d3a6c6293a..7a19f0e2ca 100644 --- a/core/src/test/java/com/netflix/conductor/core/execution/TestWorkflowRepairService.java +++ b/core/src/test/java/com/netflix/conductor/core/execution/TestWorkflowRepairService.java @@ -25,6 +25,8 @@ import org.junit.Before; import org.junit.Test; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_DECISION; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW; import static org.junit.Assert.assertFalse; import static org.junit.Assert.assertTrue; import static org.mockito.ArgumentMatchers.anyLong; @@ -128,11 +130,11 @@ public void start(Workflow workflow, Task task, WorkflowExecutor executor) { @Test public void assertSyncSystemTasksAreNotCheckedAgainstQueue() { // Return a Decision object to init WorkflowSystemTask registry. - when(systemTaskRegistry.get(Decision.NAME)).thenReturn(new Decision()); - when(systemTaskRegistry.isSystemTask(Decision.NAME)).thenReturn(true); + when(systemTaskRegistry.get(TASK_TYPE_DECISION)).thenReturn(new Decision()); + when(systemTaskRegistry.isSystemTask(TASK_TYPE_DECISION)).thenReturn(true); Task task = new Task(); - task.setTaskType(Decision.NAME); + task.setTaskType(TASK_TYPE_DECISION); task.setStatus(Task.Status.SCHEDULED); assertFalse(workflowRepairService.verifyAndRepairTask(task)); @@ -145,13 +147,13 @@ public void assertSyncSystemTasksAreNotCheckedAgainstQueue() { @Test public void assertAsyncCompleteSystemTasksAreNotCheckedAgainstQueue() { Task task = new Task(); - task.setTaskType(SubWorkflow.NAME); + task.setTaskType(TASK_TYPE_SUB_WORKFLOW); task.setStatus(Task.Status.IN_PROGRESS); task.setTaskId("abcd"); task.setCallbackAfterSeconds(60); WorkflowSystemTask workflowSystemTask = new SubWorkflow(new ObjectMapper()); - when(systemTaskRegistry.get(SubWorkflow.NAME)).thenReturn(workflowSystemTask); + when(systemTaskRegistry.get(TASK_TYPE_SUB_WORKFLOW)).thenReturn(workflowSystemTask); assertTrue(workflowSystemTask.isAsyncComplete(task)); diff --git a/core/src/test/java/com/netflix/conductor/core/execution/mapper/SubWorkflowTaskMapperTest.java b/core/src/test/java/com/netflix/conductor/core/execution/mapper/SubWorkflowTaskMapperTest.java index b0784689d5..399532ed63 100644 --- a/core/src/test/java/com/netflix/conductor/core/execution/mapper/SubWorkflowTaskMapperTest.java +++ b/core/src/test/java/com/netflix/conductor/core/execution/mapper/SubWorkflowTaskMapperTest.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.mapper; @@ -19,7 +19,6 @@ import com.netflix.conductor.common.run.Workflow; import com.netflix.conductor.core.exception.TerminateWorkflowException; import com.netflix.conductor.core.execution.DeciderService; -import com.netflix.conductor.core.execution.tasks.SubWorkflow; import com.netflix.conductor.core.utils.IDGenerator; import com.netflix.conductor.core.utils.ParametersUtils; import com.netflix.conductor.dao.MetadataDAO; @@ -32,6 +31,7 @@ import java.util.List; import java.util.Map; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW; import static org.junit.Assert.assertEquals; import static org.junit.Assert.assertFalse; import static org.mockito.ArgumentMatchers.any; @@ -98,7 +98,7 @@ public void getMappedTasks() { Task subWorkFlowTask = mappedTasks.get(0); assertEquals(Task.Status.SCHEDULED, subWorkFlowTask.getStatus()); - assertEquals(SubWorkflow.NAME, subWorkFlowTask.getTaskType()); + assertEquals(TASK_TYPE_SUB_WORKFLOW, subWorkFlowTask.getTaskType()); assertEquals(taskToDomain, subWorkFlowTask.getInputData().get("subWorkflowTaskToDomain")); } @@ -145,7 +145,7 @@ public void testTaskToDomain() { Task subWorkFlowTask = mappedTasks.get(0); assertEquals(Task.Status.SCHEDULED, subWorkFlowTask.getStatus()); - assertEquals(SubWorkflow.NAME, subWorkFlowTask.getTaskType()); + assertEquals(TASK_TYPE_SUB_WORKFLOW, subWorkFlowTask.getTaskType()); } diff --git a/core/src/test/java/com/netflix/conductor/core/execution/mapper/WaitTaskMapperTest.java b/core/src/test/java/com/netflix/conductor/core/execution/mapper/WaitTaskMapperTest.java index 38b4657371..988e7da241 100644 --- a/core/src/test/java/com/netflix/conductor/core/execution/mapper/WaitTaskMapperTest.java +++ b/core/src/test/java/com/netflix/conductor/core/execution/mapper/WaitTaskMapperTest.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.mapper; @@ -18,7 +18,6 @@ import com.netflix.conductor.common.metadata.workflow.WorkflowDef; import com.netflix.conductor.common.metadata.workflow.WorkflowTask; import com.netflix.conductor.common.run.Workflow; -import com.netflix.conductor.core.execution.tasks.Wait; import com.netflix.conductor.core.utils.IDGenerator; import com.netflix.conductor.core.utils.ParametersUtils; import org.junit.Test; @@ -26,6 +25,7 @@ import java.util.HashMap; import java.util.List; +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_WAIT; import static org.junit.Assert.assertEquals; import static org.mockito.Mockito.mock; @@ -61,6 +61,6 @@ public void getMappedTasks() { //Then assertEquals(1, mappedTasks.size()); - assertEquals(Wait.NAME, mappedTasks.get(0).getTaskType()); + assertEquals(TASK_TYPE_WAIT, mappedTasks.get(0).getTaskType()); } } diff --git a/core/src/test/java/com/netflix/conductor/core/execution/tasks/TestSystemTaskWorkerCoordinator.java b/core/src/test/java/com/netflix/conductor/core/execution/tasks/TestSystemTaskWorkerCoordinator.java index e4e4e83628..6a1b9a4c8d 100644 --- a/core/src/test/java/com/netflix/conductor/core/execution/tasks/TestSystemTaskWorkerCoordinator.java +++ b/core/src/test/java/com/netflix/conductor/core/execution/tasks/TestSystemTaskWorkerCoordinator.java @@ -1,14 +1,14 @@ /* - * Copyright 2020 Netflix, Inc. - *

- * Licensed 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. + * Copyright 2021 Netflix, Inc. + *

+ * Licensed 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 com.netflix.conductor.core.execution.tasks; @@ -16,17 +16,15 @@ import com.netflix.conductor.core.execution.WorkflowExecutor; import com.netflix.conductor.dao.QueueDAO; import com.netflix.conductor.service.ExecutionService; -import java.time.Duration; import org.junit.Before; import org.junit.Test; -import java.util.Collections; +import java.time.Duration; import static org.junit.Assert.assertFalse; import static org.junit.Assert.assertTrue; import static org.mockito.Mockito.doReturn; import static org.mockito.Mockito.mock; -import static org.mockito.Mockito.spy; import static org.mockito.Mockito.when; public class TestSystemTaskWorkerCoordinator { @@ -54,7 +52,7 @@ public void setUp() { public void isSystemTask() { createTaskMapping(); SystemTaskWorkerCoordinator systemTaskWorkerCoordinator = new SystemTaskWorkerCoordinator(queueDAO, - workflowExecutor, properties, executionService, Collections.emptyList()); + workflowExecutor, properties, executionService, null); assertTrue(systemTaskWorkerCoordinator.isAsyncSystemTask(TEST_QUEUE + ISOLATION_CONSTANT)); } @@ -62,7 +60,7 @@ public void isSystemTask() { public void isSystemTaskNotPresent() { createTaskMapping(); SystemTaskWorkerCoordinator systemTaskWorkerCoordinator = new SystemTaskWorkerCoordinator(queueDAO, - workflowExecutor, properties, executionService, Collections.emptyList()); + workflowExecutor, properties, executionService, null); assertFalse(systemTaskWorkerCoordinator.isAsyncSystemTask(null)); } @@ -70,7 +68,7 @@ public void isSystemTaskNotPresent() { public void testIsFromCoordinatorExecutionNameSpace() { doReturn("exeNS").when(properties).getSystemTaskWorkerExecutionNamespace(); SystemTaskWorkerCoordinator systemTaskWorkerCoordinator = new SystemTaskWorkerCoordinator(queueDAO, - workflowExecutor, properties, executionService, Collections.emptyList()); + workflowExecutor, properties, executionService, null); assertTrue( systemTaskWorkerCoordinator.isFromCoordinatorExecutionNameSpace(TEST_QUEUE + EXECUTION_NAMESPACE_CONSTANT)); } @@ -79,6 +77,6 @@ private void createTaskMapping() { WorkflowSystemTask mockWorkflowTask = mock(WorkflowSystemTask.class); when(mockWorkflowTask.getName()).thenReturn(TEST_QUEUE); when(mockWorkflowTask.isAsync()).thenReturn(true); - SystemTaskWorkerCoordinator.taskNameWorkflowTaskMapping.put(TEST_QUEUE, mockWorkflowTask); +// SystemTaskWorkerCoordinator.taskNameWorkflowTaskMapping.put(TEST_QUEUE, mockWorkflowTask); } } diff --git a/test-harness/src/test/groovy/com/netflix/conductor/test/integration/HierarchicalForkJoinSubworkflowRestartSpec.groovy b/test-harness/src/test/groovy/com/netflix/conductor/test/integration/HierarchicalForkJoinSubworkflowRestartSpec.groovy index 32178a8f18..2358ad8a03 100644 --- a/test-harness/src/test/groovy/com/netflix/conductor/test/integration/HierarchicalForkJoinSubworkflowRestartSpec.groovy +++ b/test-harness/src/test/groovy/com/netflix/conductor/test/integration/HierarchicalForkJoinSubworkflowRestartSpec.groovy @@ -16,13 +16,13 @@ package com.netflix.conductor.test.integration import com.netflix.conductor.common.metadata.tasks.Task import com.netflix.conductor.common.metadata.tasks.TaskDef import com.netflix.conductor.common.run.Workflow -import com.netflix.conductor.core.execution.tasks.Fork import com.netflix.conductor.core.execution.tasks.SubWorkflow import com.netflix.conductor.dao.QueueDAO import com.netflix.conductor.test.base.AbstractSpecification import org.springframework.beans.factory.annotation.Autowired import spock.lang.Shared +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_FORK import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_JOIN import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW import static com.netflix.conductor.test.util.WorkflowTestUtil.verifyPolledAndAcknowledgedTask @@ -78,7 +78,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.SCHEDULED @@ -110,7 +110,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.SCHEDULED @@ -159,7 +159,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.FAILED tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.FAILED @@ -176,7 +176,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.FAILED tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.FAILED @@ -210,7 +210,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.SCHEDULED @@ -233,7 +233,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(newMidLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.SCHEDULED @@ -304,7 +304,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.SCHEDULED @@ -318,7 +318,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -396,7 +396,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -411,7 +411,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -430,7 +430,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -445,7 +445,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -484,7 +484,7 @@ class HierarchicalForkJoinSubworkflowRestartSpec extends AbstractSpecification { assert with(workflowExecutionService.getExecutionStatus(workflowId, true)) { status == Workflow.WorkflowStatus.COMPLETED tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.COMPLETED diff --git a/test-harness/src/test/groovy/com/netflix/conductor/test/integration/HierarchicalForkJoinSubworkflowRetrySpec.groovy b/test-harness/src/test/groovy/com/netflix/conductor/test/integration/HierarchicalForkJoinSubworkflowRetrySpec.groovy index 2819e53ce3..54f1307591 100644 --- a/test-harness/src/test/groovy/com/netflix/conductor/test/integration/HierarchicalForkJoinSubworkflowRetrySpec.groovy +++ b/test-harness/src/test/groovy/com/netflix/conductor/test/integration/HierarchicalForkJoinSubworkflowRetrySpec.groovy @@ -16,13 +16,13 @@ package com.netflix.conductor.test.integration import com.netflix.conductor.common.metadata.tasks.Task import com.netflix.conductor.common.metadata.tasks.TaskDef import com.netflix.conductor.common.run.Workflow -import com.netflix.conductor.core.execution.tasks.Fork import com.netflix.conductor.core.execution.tasks.SubWorkflow import com.netflix.conductor.dao.QueueDAO import com.netflix.conductor.test.base.AbstractSpecification import org.springframework.beans.factory.annotation.Autowired import spock.lang.Shared +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_FORK import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_JOIN import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW import static com.netflix.conductor.test.util.WorkflowTestUtil.verifyPolledAndAcknowledgedTask @@ -78,7 +78,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.SCHEDULED @@ -110,7 +110,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.SCHEDULED @@ -159,7 +159,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.FAILED tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.FAILED @@ -176,7 +176,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.FAILED tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.FAILED @@ -210,7 +210,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 5 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.FAILED @@ -234,7 +234,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(newMidLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.SCHEDULED @@ -305,7 +305,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 5 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.FAILED @@ -323,7 +323,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -404,7 +404,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -419,7 +419,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -438,7 +438,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -453,7 +453,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -509,7 +509,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -524,7 +524,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -555,7 +555,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -570,7 +570,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -625,7 +625,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -640,7 +640,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -671,7 +671,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -686,7 +686,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -753,7 +753,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -768,7 +768,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -787,7 +787,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(midLevelWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -802,7 +802,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(rootWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.IN_PROGRESS @@ -845,7 +845,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { assert with(workflowExecutionService.getExecutionStatus(workflowId, true)) { status == Workflow.WorkflowStatus.COMPLETED tasks.size() == 4 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.COMPLETED @@ -861,7 +861,7 @@ class HierarchicalForkJoinSubworkflowRetrySpec extends AbstractSpecification { assert with(workflowExecutionService.getExecutionStatus(workflowId, true)) { status == Workflow.WorkflowStatus.COMPLETED tasks.size() == 5 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED tasks[1].taskType == TASK_TYPE_SUB_WORKFLOW tasks[1].status == Task.Status.FAILED diff --git a/test-harness/src/test/groovy/com/netflix/conductor/test/integration/NestedForkJoinSubWorkflowSpec.groovy b/test-harness/src/test/groovy/com/netflix/conductor/test/integration/NestedForkJoinSubWorkflowSpec.groovy index 76cd1c4c0d..ce1513f232 100644 --- a/test-harness/src/test/groovy/com/netflix/conductor/test/integration/NestedForkJoinSubWorkflowSpec.groovy +++ b/test-harness/src/test/groovy/com/netflix/conductor/test/integration/NestedForkJoinSubWorkflowSpec.groovy @@ -16,13 +16,13 @@ package com.netflix.conductor.test.integration import com.netflix.conductor.common.metadata.tasks.Task import com.netflix.conductor.common.metadata.tasks.TaskDef import com.netflix.conductor.common.run.Workflow -import com.netflix.conductor.core.execution.tasks.Fork import com.netflix.conductor.core.execution.tasks.SubWorkflow import com.netflix.conductor.dao.QueueDAO import com.netflix.conductor.test.base.AbstractSpecification import org.springframework.beans.factory.annotation.Autowired import spock.lang.Shared +import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_FORK import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_JOIN import static com.netflix.conductor.common.metadata.tasks.TaskType.TASK_TYPE_SUB_WORKFLOW @@ -76,9 +76,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.SCHEDULED @@ -105,9 +105,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(parentWorkflowInstance) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.IN_PROGRESS @@ -150,9 +150,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.FAILED tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.FAILED @@ -197,9 +197,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.IN_PROGRESS @@ -221,9 +221,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.IN_PROGRESS @@ -256,9 +256,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.COMPLETED @@ -297,9 +297,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.SCHEDULED @@ -324,9 +324,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.IN_PROGRESS @@ -368,9 +368,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.COMPLETED @@ -409,9 +409,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 8 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.FAILED @@ -438,9 +438,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 8 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.FAILED @@ -486,9 +486,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 8 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.FAILED @@ -513,9 +513,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.COMPLETED tasks.size() == 8 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.FAILED @@ -565,9 +565,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.IN_PROGRESS @@ -589,9 +589,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.IN_PROGRESS @@ -627,9 +627,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.COMPLETED @@ -681,9 +681,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.IN_PROGRESS @@ -705,9 +705,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.IN_PROGRESS @@ -743,9 +743,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.RUNNING tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.COMPLETED @@ -770,9 +770,9 @@ class NestedForkJoinSubWorkflowSpec extends AbstractSpecification { assert with(workflowExecutionService.getExecutionStatus(parentWorkflowId, true)) { status == Workflow.WorkflowStatus.COMPLETED tasks.size() == 7 - tasks[0].taskType == Fork.NAME + tasks[0].taskType == TASK_TYPE_FORK tasks[0].status == Task.Status.COMPLETED - tasks[1].taskType == Fork.NAME + tasks[1].taskType == TASK_TYPE_FORK tasks[1].status == Task.Status.COMPLETED tasks[2].taskType == TASK_TYPE_SUB_WORKFLOW tasks[2].status == Task.Status.COMPLETED