Skip to content

Commit

Permalink
[Fix-16705] [Built-in Parameter] Fix project name and definition name…
Browse files Browse the repository at this point in the history
… variable parameter not resolved (#16715)
  • Loading branch information
sdhzwc authored Oct 25, 2024
1 parent cf9d487 commit 5f319e5
Show file tree
Hide file tree
Showing 16 changed files with 186 additions and 9 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -21,10 +21,12 @@
import static org.apache.dolphinscheduler.common.utils.JSONUtils.parseObject;

import org.apache.dolphinscheduler.dao.entity.Command;
import org.apache.dolphinscheduler.dao.entity.Project;
import org.apache.dolphinscheduler.dao.entity.TaskDefinition;
import org.apache.dolphinscheduler.dao.entity.TaskInstance;
import org.apache.dolphinscheduler.dao.entity.WorkflowDefinition;
import org.apache.dolphinscheduler.dao.entity.WorkflowInstance;
import org.apache.dolphinscheduler.dao.repository.ProjectDao;
import org.apache.dolphinscheduler.dao.repository.TaskInstanceDao;
import org.apache.dolphinscheduler.dao.repository.WorkflowDefinitionLogDao;
import org.apache.dolphinscheduler.extract.master.command.ICommandParam;
Expand Down Expand Up @@ -64,12 +66,16 @@ public abstract class AbstractCommandHandler implements ICommandHandler {
@Autowired
protected List<IWorkflowLifecycleListener> workflowLifecycleListeners;

@Autowired
protected ProjectDao projectDao;

@Override
public WorkflowExecutionRunnable handleCommand(final Command command) {
final WorkflowExecuteContextBuilder workflowExecuteContextBuilder = WorkflowExecuteContext.builder()
.withCommand(command);

assembleWorkflowDefinition(workflowExecuteContextBuilder);
assembleProject(workflowExecuteContextBuilder);
assembleWorkflowGraph(workflowExecuteContextBuilder);
assembleWorkflowInstance(workflowExecuteContextBuilder);
assembleWorkflowInstanceLifecycleListeners(workflowExecuteContextBuilder);
Expand Down Expand Up @@ -146,4 +152,12 @@ protected List<TaskInstance> getValidTaskInstance(final WorkflowInstance workflo
workflowInstance.getTestFlag());
}

protected void assembleProject(
final WorkflowExecuteContextBuilder workflowExecuteContextBuilder) {
final WorkflowDefinition workflowDefinition = workflowExecuteContextBuilder.getWorkflowDefinition();
final Project project = projectDao.queryByCode(workflowDefinition.getProjectCode());
checkArgument(project != null, "Cannot find the project code: " + workflowDefinition.getProjectCode());
workflowExecuteContextBuilder.setProject(project);
}

}
Original file line number Diff line number Diff line change
Expand Up @@ -123,6 +123,7 @@ protected void assembleWorkflowExecutionGraph(final WorkflowExecuteContextBuilde
.builder()
.workflowExecutionGraph(workflowExecutionGraph)
.workflowDefinition(workflowExecuteContextBuilder.getWorkflowDefinition())
.project(workflowExecuteContextBuilder.getProject())
.workflowInstance(workflowExecuteContextBuilder.getWorkflowInstance())
.taskDefinition(workflowGraph.getTaskNodeByName(task))
.taskInstance(taskInstanceMap.get(task))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -98,6 +98,7 @@ protected void assembleWorkflowExecutionGraph(final WorkflowExecuteContextBuilde
.builder()
.workflowExecutionGraph(workflowExecutionGraph)
.workflowDefinition(workflowExecuteContextBuilder.getWorkflowDefinition())
.project(workflowExecuteContextBuilder.getProject())
.workflowInstance(workflowExecuteContextBuilder.getWorkflowInstance())
.taskDefinition(workflowGraph.getTaskNodeByName(task))
.workflowEventBus(workflowExecuteContextBuilder.getWorkflowEventBus())
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -119,6 +119,7 @@ protected void assembleWorkflowExecutionGraph(final WorkflowExecuteContextBuilde
.builder()
.workflowExecutionGraph(workflowExecutionGraph)
.workflowDefinition(workflowExecuteContextBuilder.getWorkflowDefinition())
.project(workflowExecuteContextBuilder.getProject())
.workflowInstance(workflowExecuteContextBuilder.getWorkflowInstance())
.taskDefinition(workflowGraph.getTaskNodeByName(task))
.taskInstance(taskInstanceMap.get(task))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@

package org.apache.dolphinscheduler.server.master.engine.task.runnable;

import org.apache.dolphinscheduler.dao.entity.Project;
import org.apache.dolphinscheduler.dao.entity.TaskDefinition;
import org.apache.dolphinscheduler.dao.entity.TaskInstance;
import org.apache.dolphinscheduler.dao.entity.WorkflowDefinition;
Expand All @@ -35,5 +36,6 @@ public class TaskExecutionContextCreateRequest {
private WorkflowInstance workflowInstance;
private TaskDefinition taskDefinition;
private TaskInstance taskInstance;
private Project project;

}
Original file line number Diff line number Diff line change
Expand Up @@ -20,6 +20,7 @@
import static com.google.common.base.Preconditions.checkNotNull;
import static com.google.common.base.Preconditions.checkState;

import org.apache.dolphinscheduler.dao.entity.Project;
import org.apache.dolphinscheduler.dao.entity.TaskDefinition;
import org.apache.dolphinscheduler.dao.entity.TaskInstance;
import org.apache.dolphinscheduler.dao.entity.WorkflowDefinition;
Expand Down Expand Up @@ -57,6 +58,8 @@ public class TaskExecutionRunnable implements ITaskExecutionRunnable {
@Getter
private final WorkflowDefinition workflowDefinition;
@Getter
private final Project project;
@Getter
private final WorkflowInstance workflowInstance;
@Getter
private TaskInstance taskInstance;
Expand All @@ -70,6 +73,7 @@ public TaskExecutionRunnable(TaskExecutionRunnableBuilder taskExecutionRunnableB
this.workflowExecutionGraph = checkNotNull(taskExecutionRunnableBuilder.getWorkflowExecutionGraph());
this.workflowEventBus = checkNotNull(taskExecutionRunnableBuilder.getWorkflowEventBus());
this.workflowDefinition = checkNotNull(taskExecutionRunnableBuilder.getWorkflowDefinition());
this.project = checkNotNull(taskExecutionRunnableBuilder.getProject());
this.workflowInstance = checkNotNull(taskExecutionRunnableBuilder.getWorkflowInstance());
this.taskDefinition = checkNotNull(taskExecutionRunnableBuilder.getTaskDefinition());
this.taskInstance = taskExecutionRunnableBuilder.getTaskInstance();
Expand Down Expand Up @@ -144,6 +148,7 @@ private void initializeTaskExecutionContext() {
checkState(isTaskInstanceInitialized(), "The task instance is null, can't initialize TaskExecutionContext.");
final TaskExecutionContextCreateRequest request = TaskExecutionContextCreateRequest.builder()
.workflowDefinition(workflowDefinition)
.project(project)
.workflowInstance(workflowInstance)
.taskDefinition(taskDefinition)
.taskInstance(taskInstance)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@

package org.apache.dolphinscheduler.server.master.engine.task.runnable;

import org.apache.dolphinscheduler.dao.entity.Project;
import org.apache.dolphinscheduler.dao.entity.TaskDefinition;
import org.apache.dolphinscheduler.dao.entity.TaskInstance;
import org.apache.dolphinscheduler.dao.entity.WorkflowDefinition;
Expand All @@ -37,6 +38,7 @@ public class TaskExecutionRunnableBuilder {

private final IWorkflowExecutionGraph workflowExecutionGraph;
private final WorkflowDefinition workflowDefinition;
private final Project project;
private final WorkflowInstance workflowInstance;
private final TaskDefinition taskDefinition;
private final TaskInstance taskInstance;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -36,7 +36,9 @@
import org.apache.dolphinscheduler.dao.entity.DqRule;
import org.apache.dolphinscheduler.dao.entity.DqRuleExecuteSql;
import org.apache.dolphinscheduler.dao.entity.DqRuleInputEntry;
import org.apache.dolphinscheduler.dao.entity.Project;
import org.apache.dolphinscheduler.dao.entity.TaskInstance;
import org.apache.dolphinscheduler.dao.entity.WorkflowDefinition;
import org.apache.dolphinscheduler.dao.entity.WorkflowInstance;
import org.apache.dolphinscheduler.plugin.task.api.DataQualityTaskExecutionContext;
import org.apache.dolphinscheduler.plugin.task.api.K8sTaskExecutionContext;
Expand Down Expand Up @@ -98,6 +100,8 @@ public class TaskExecutionContextFactory {
public TaskExecutionContext createTaskExecutionContext(TaskExecutionContextCreateRequest request) {
TaskInstance taskInstance = request.getTaskInstance();
WorkflowInstance workflowInstance = request.getWorkflowInstance();
WorkflowDefinition workflowDefinition = request.getWorkflowDefinition();
Project project = request.getProject();

ResourceParametersHelper resources = TaskPluginManager.getTaskChannel(taskInstance.getTaskType())
.parseParameters(taskInstance.getTaskParams())
Expand All @@ -108,8 +112,10 @@ public TaskExecutionContext createTaskExecutionContext(TaskExecutionContextCreat

AbstractParameters baseParam =
TaskPluginManager.parseTaskParameters(taskInstance.getTaskType(), taskInstance.getTaskParams());

Map<String, Property> propertyMap =
curingParamsService.paramParsingPreparation(taskInstance, baseParam, workflowInstance);
curingParamsService.paramParsingPreparation(taskInstance, baseParam, workflowInstance,
project.getName(), workflowDefinition.getName());
TaskExecutionContext taskExecutionContext = TaskExecutionContextBuilder.get()
.buildWorkflowInstanceHost(masterConfig.getMasterAddress())
.buildTaskInstanceRelatedInfo(taskInstance)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@
package org.apache.dolphinscheduler.server.master.runner;

import org.apache.dolphinscheduler.dao.entity.Command;
import org.apache.dolphinscheduler.dao.entity.Project;
import org.apache.dolphinscheduler.dao.entity.WorkflowDefinition;
import org.apache.dolphinscheduler.dao.entity.WorkflowInstance;
import org.apache.dolphinscheduler.server.master.engine.WorkflowEventBus;
Expand All @@ -40,6 +41,8 @@ public class WorkflowExecuteContext implements IWorkflowExecuteContext {

private final WorkflowDefinition workflowDefinition;

private final Project project;

private final WorkflowInstance workflowInstance;

private final IWorkflowGraph workflowGraph;
Expand Down Expand Up @@ -72,6 +75,8 @@ public static class WorkflowExecuteContextBuilder {

private List<IWorkflowLifecycleListener> workflowInstanceLifecycleListeners;

private Project project;

public WorkflowExecuteContextBuilder withCommand(Command command) {
this.command = command;
return this;
Expand All @@ -81,6 +86,7 @@ public WorkflowExecuteContext build() {
return new WorkflowExecuteContext(
command,
workflowDefinition,
project,
workflowInstance,
workflowGraph,
workflowExecutionGraph,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -709,4 +709,40 @@ public void testStartWorkflowFromStartNodes_with_threeParallelSuccessTask() {

assertThat(workflowRepository.getAll()).isEmpty();
}

@Test
@DisplayName("Test start a workflow which using workflow built in params")
public void testStartWorkflow_usingWorkflowBuiltInParam() {
final String yaml = "/it/start/workflow_with_built_in_param.yaml";
final WorkflowTestCaseContext context = workflowTestCaseContextFactory.initializeContextFromYaml(yaml);
final WorkflowDefinition workflow = context.getWorkflows().get(0);

final WorkflowOperator.WorkflowTriggerDTO workflowTriggerDTO = WorkflowOperator.WorkflowTriggerDTO.builder()
.workflowDefinition(workflow)
.runWorkflowCommandParam(new RunWorkflowCommandParam())
.build();
workflowOperator.manualTriggerWorkflow(workflowTriggerDTO);

await()
.atMost(Duration.ofMinutes(1))
.untilAsserted(() -> {
Assertions
.assertThat(repository.queryWorkflowInstance(workflow))
.satisfiesExactly(workflowInstance -> assertThat(workflowInstance.getState())
.isEqualTo(WorkflowExecutionStatus.SUCCESS));
Assertions
.assertThat(repository.queryTaskInstance(workflow))
.hasSize(2)
.anySatisfy(taskInstance -> {
assertThat(taskInstance.getName()).isEqualTo("A");
assertThat(taskInstance.getState()).isEqualTo(TaskExecutionStatus.SUCCESS);
})
.anySatisfy(taskInstance -> {
assertThat(taskInstance.getName()).isEqualTo("B");
assertThat(taskInstance.getState()).isEqualTo(TaskExecutionStatus.SUCCESS);
});
});

assertThat(workflowRepository.getAll()).isEmpty();
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -28,6 +28,7 @@
import static org.mockito.Mockito.when;

import org.apache.dolphinscheduler.common.utils.JSONUtils;
import org.apache.dolphinscheduler.dao.entity.Project;
import org.apache.dolphinscheduler.dao.entity.TaskDefinition;
import org.apache.dolphinscheduler.dao.entity.TaskInstance;
import org.apache.dolphinscheduler.dao.entity.WorkflowDefinition;
Expand Down Expand Up @@ -118,6 +119,7 @@ private ITaskExecutionRunnable createTaskExecuteRunnable(final TaskInstance task
.taskInstance(taskInstance)
.workflowExecutionGraph(new WorkflowExecutionGraph())
.workflowDefinition(new WorkflowDefinition())
.project(new Project())
.taskDefinition(new TaskDefinition())
.workflowEventBus(new WorkflowEventBus())
.build();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -24,6 +24,7 @@
import static org.mockito.Mockito.when;

import org.apache.dolphinscheduler.common.enums.Priority;
import org.apache.dolphinscheduler.dao.entity.Project;
import org.apache.dolphinscheduler.dao.entity.TaskDefinition;
import org.apache.dolphinscheduler.dao.entity.TaskInstance;
import org.apache.dolphinscheduler.dao.entity.WorkflowDefinition;
Expand Down Expand Up @@ -194,6 +195,7 @@ private ITaskExecutionRunnable createTaskExecuteRunnable() {
.taskInstance(taskInstance)
.workflowExecutionGraph(new WorkflowExecutionGraph())
.workflowDefinition(new WorkflowDefinition())
.project(new Project())
.taskDefinition(new TaskDefinition())
.workflowEventBus(new WorkflowEventBus())
.build();
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,81 @@
#
# Licensed to the Apache Software Foundation (ASF) under one or more
# contributor license agreements. See the NOTICE file distributed with
# this work for additional information regarding copyright ownership.
# The ASF licenses this file to You under the Apache License, Version 2.0
# (the "License"); you may not use this file except in compliance with
# the License. You may obtain a copy of the License at
#
# http://www.apache.org/licenses/LICENSE-2.0
#
# Unless required by applicable law or agreed to in writing, software
# distributed under the License is distributed on an "AS IS" BASIS,
# WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
# See the License for the specific language governing permissions and
# limitations under the License.
#

project:
name: MasterIntegrationTest
code: 1
description: This is a fake project
userId: 1
userName: admin
createTime: 2024-08-12 00:00:00
updateTime: 2021-08-12 00:00:00

workflows:
- name: workflow_with_one_fake_task_success
code: 1
version: 1
projectCode: 1
description: This is a fake workflow with single task
releaseState: ONLINE
createTime: 2024-08-12 00:00:00
updateTime: 2021-08-12 00:00:00
userId: 1
executionType: PARALLEL

tasks:
- name: A
code: 1
version: 1
projectCode: 1
userId: 1
taskType: LogicFakeTask
taskParams: '{"localParams":[],"shellScript":"if [ \"${system.project.name}\" = \"MasterIntegrationTest\" ]; then\n exit 0 \nelse\n exit 1\nfi","resourceList":[]}'
workerGroup: default
createTime: 2024-08-12 00:00:00
updateTime: 2021-08-12 00:00:00
taskExecuteType: BATCH
- name: B
code: 2
version: 1
projectCode: 1
userId: 1
taskType: LogicFakeTask
taskParams: '{"localParams":[],"shellScript":"if [ \"${system.workflow.definition.name}\" = \"workflow_with_one_fake_task_success\" ]; then\n exit 0 \nelse\n exit 1\nfi","resourceList":[]}'
workerGroup: default
createTime: 2024-08-12 00:00:00
updateTime: 2021-08-12 00:00:00
taskExecuteType: BATCH

taskRelations:
- projectCode: 1
workflowDefinitionCode: 1
workflowDefinitionVersion: 1
preTaskCode: 0
preTaskVersion: 0
postTaskCode: 1
postTaskVersion: 1
createTime: 2024-08-12 00:00:00
updateTime: 2024-08-12 00:00:00
- projectCode: 1
workflowDefinitionCode: 1
workflowDefinitionVersion: 1
preTaskCode: 0
preTaskVersion: 0
postTaskCode: 2
postTaskVersion: 1
createTime: 2024-08-12 00:00:00
updateTime: 2024-08-12 00:00:00
Original file line number Diff line number Diff line change
Expand Up @@ -76,11 +76,15 @@ String curingGlobalParams(Integer workflowInstanceId, Map<String, String> global
* @param parameters
* @param taskInstance
* @param workflowInstance
* @param projectName
* @param workflowDefinitionName
* @return
*/
Map<String, Property> paramParsingPreparation(@NonNull TaskInstance taskInstance,
@NonNull AbstractParameters parameters,
@NonNull WorkflowInstance workflowInstance);
@NonNull WorkflowInstance workflowInstance,
String projectName,
String workflowDefinitionName);

/**
* Parse workflow star parameter
Expand Down
Loading

0 comments on commit 5f319e5

Please sign in to comment.