From 0bfb0e4b164c8a9cf26ee84fb3a4a9bf6a4bb3a3 Mon Sep 17 00:00:00 2001 From: "DK.Pino" Date: Thu, 2 Jan 2020 17:14:34 +0800 Subject: [PATCH 01/40] remove lombok depend in common dodule (#1677) --- dolphinscheduler-common/pom.xml | 7 ------- 1 file changed, 7 deletions(-) diff --git a/dolphinscheduler-common/pom.xml b/dolphinscheduler-common/pom.xml index 10cede7988..955e956251 100644 --- a/dolphinscheduler-common/pom.xml +++ b/dolphinscheduler-common/pom.xml @@ -600,13 +600,6 @@ compile - - org.projectlombok - lombok - ${lombok.version} - compile - - org.springframework spring-context From 5949c8d6fbdf58e6ab4f35150fef7dbc5a73e47c Mon Sep 17 00:00:00 2001 From: loushang Date: Thu, 2 Jan 2020 17:20:57 +0800 Subject: [PATCH 02/40] fix SessionService signOut method exception bug --- .../apache/dolphinscheduler/api/service/SessionService.java | 4 ---- 1 file changed, 4 deletions(-) diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/SessionService.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/SessionService.java index 89478d3e0e..f3f66d730d 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/SessionService.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/SessionService.java @@ -139,7 +139,6 @@ public class SessionService extends BaseService{ * @param loginUser login user */ public void signOut(String ip, User loginUser) { - try { /** * query session by user id and ip */ @@ -147,8 +146,5 @@ public class SessionService extends BaseService{ //delete session sessionMapper.deleteById(session.getId()); - }catch (Exception e){ - - } } } From 30dd867acbc12a46dbe181d507b4510d819e4079 Mon Sep 17 00:00:00 2001 From: "DK.Pino" Date: Thu, 2 Jan 2020 17:26:02 +0800 Subject: [PATCH 03/40] remove lombok depend in dao dodule (#1683) --- .../apache/dolphinscheduler/dao/entity/ProcessDefinition.java | 2 -- .../java/org/apache/dolphinscheduler/dao/entity/Schedule.java | 2 -- .../org/apache/dolphinscheduler/dao/entity/TaskInstance.java | 2 -- .../main/java/org/apache/dolphinscheduler/dao/entity/User.java | 2 -- .../org/apache/dolphinscheduler/dao/entity/UserAlertGroup.java | 2 -- .../org/apache/dolphinscheduler/dao/entity/WorkerGroup.java | 2 -- 6 files changed, 12 deletions(-) diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/ProcessDefinition.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/ProcessDefinition.java index d3de4123b0..cd0494ecc6 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/ProcessDefinition.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/ProcessDefinition.java @@ -25,7 +25,6 @@ import com.baomidou.mybatisplus.annotation.TableField; import com.baomidou.mybatisplus.annotation.TableId; import com.baomidou.mybatisplus.annotation.TableName; import com.baomidou.mybatisplus.core.toolkit.StringUtils; -import lombok.Data; import java.util.Date; import java.util.List; @@ -36,7 +35,6 @@ import java.util.stream.Collectors; /** * process definition */ -@Data @TableName("t_ds_process_definition") public class ProcessDefinition { /** diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/Schedule.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/Schedule.java index 65ae4d8e12..cfda49df6e 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/Schedule.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/Schedule.java @@ -20,7 +20,6 @@ import com.baomidou.mybatisplus.annotation.IdType; import com.baomidou.mybatisplus.annotation.TableField; import com.baomidou.mybatisplus.annotation.TableId; import com.baomidou.mybatisplus.annotation.TableName; -import lombok.Data; import org.apache.dolphinscheduler.common.enums.FailureStrategy; import org.apache.dolphinscheduler.common.enums.Priority; import org.apache.dolphinscheduler.common.enums.ReleaseState; @@ -32,7 +31,6 @@ import java.util.Date; * schedule * */ -@Data @TableName("t_ds_schedules") public class Schedule { diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/TaskInstance.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/TaskInstance.java index 3492fe5b17..2db1eda8f4 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/TaskInstance.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/TaskInstance.java @@ -26,14 +26,12 @@ import org.apache.dolphinscheduler.common.utils.JSONUtils; import com.baomidou.mybatisplus.annotation.IdType; import com.baomidou.mybatisplus.annotation.TableId; import com.baomidou.mybatisplus.annotation.TableName; -import lombok.Data; import java.util.Date; /** * task instance */ -@Data @TableName("t_ds_task_instance") public class TaskInstance { diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/User.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/User.java index b6def18922..8849be0b83 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/User.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/User.java @@ -22,14 +22,12 @@ import com.baomidou.mybatisplus.annotation.IdType; import com.baomidou.mybatisplus.annotation.TableField; import com.baomidou.mybatisplus.annotation.TableId; import com.baomidou.mybatisplus.annotation.TableName; -import lombok.Data; import java.util.Date; /** * user */ -@Data @TableName("t_ds_user") public class User { diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/UserAlertGroup.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/UserAlertGroup.java index 84742468b6..e86b32e075 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/UserAlertGroup.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/UserAlertGroup.java @@ -20,14 +20,12 @@ import com.baomidou.mybatisplus.annotation.IdType; import com.baomidou.mybatisplus.annotation.TableField; import com.baomidou.mybatisplus.annotation.TableId; import com.baomidou.mybatisplus.annotation.TableName; -import lombok.Data; import java.util.Date; /** * user alert group */ -@Data @TableName("t_ds_relation_user_alertgroup") public class UserAlertGroup { diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/WorkerGroup.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/WorkerGroup.java index 5c4595b1cf..a732dbbe6e 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/WorkerGroup.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/entity/WorkerGroup.java @@ -19,14 +19,12 @@ package org.apache.dolphinscheduler.dao.entity; import com.baomidou.mybatisplus.annotation.IdType; import com.baomidou.mybatisplus.annotation.TableId; import com.baomidou.mybatisplus.annotation.TableName; -import lombok.Data; import java.util.Date; /** * worker group for task running */ -@Data @TableName("t_ds_worker_group") public class WorkerGroup { From 26ed786c4d4cd1b8ae0e49a4a2ac7f2677a1948a Mon Sep 17 00:00:00 2001 From: Yelli <51317527+Yeleights@users.noreply.github.com> Date: Thu, 2 Jan 2020 19:45:05 +0800 Subject: [PATCH 04/40] #1544 fix bug: workflow import (#1676) * modify FileUtils.readFile2Str * #1300 Add right alignment function in sql email content * cancel formatted for alert_mail_template.ftl * #747 sql task password Log desensitization * cancel mail_temple * edit ExcelUtils * modify test method name * #747 sql task password Log desensitization * #1544 workflow import * Constants add DATASOURCE_PASSWORD_REGEX * #747 sql task password Log desensitization * deal with import project have sub process * modify export process addTaskNodeParam method name * add testAddTaskNodeSpecialParam UT * add ProcessDefinitionServiceTest-ut to pom * add testImportSubProcess in ProcessDefinitionServiceTest * add testImportSubProcess in ProcessDefinitionServiceTest * add testImportProcessDefinition --- .../ProcessDefinitionController.java | 2 +- .../api/controller/ProjectController.java | 9 +- .../api/service/ProcessDefinitionService.java | 486 ++++++++++++------ .../service/ProcessDefinitionServiceTest.java | 294 ++++++++++- .../fileUpdate/definitionUpdate.vue | 1 + pom.xml | 1 + 6 files changed, 628 insertions(+), 165 deletions(-) diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/ProcessDefinitionController.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/ProcessDefinitionController.java index 275dfdd3db..f2e4c4d8d2 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/ProcessDefinitionController.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/ProcessDefinitionController.java @@ -460,7 +460,7 @@ public class ProcessDefinitionController extends BaseController{ } } - if(deleteFailedIdList.size() > 0){ + if(!deleteFailedIdList.isEmpty()){ putMsg(result, Status.BATCH_DELETE_PROCESS_DEFINE_BY_IDS_ERROR,StringUtils.join(deleteFailedIdList.toArray(),",")); }else{ putMsg(result, Status.SUCCESS); diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/ProjectController.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/ProjectController.java index 66e065ee48..bc015c28aa 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/ProjectController.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/ProjectController.java @@ -267,11 +267,12 @@ public class ProjectController extends BaseController { }) @PostMapping(value="/import-definition") public Result importProcessDefinition(@ApiIgnore @RequestAttribute(value = Constants.SESSION_USER) User loginUser, - @RequestParam("file") MultipartFile file){ + @RequestParam("file") MultipartFile file, + @RequestParam("projectName") String projectName){ try{ - logger.info("import process definition by id, login user:{}", - loginUser.getUserName()); - Map result = processDefinitionService.importProcessDefinition(loginUser,file); + logger.info("import process definition by id, login user:{}, project: {}", + loginUser.getUserName(), projectName); + Map result = processDefinitionService.importProcessDefinition(loginUser, file, projectName); return returnDataList(result); }catch (Exception e){ logger.error(IMPORT_PROCESS_DEFINE_ERROR.getMsg(),e); diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/ProcessDefinitionService.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/ProcessDefinitionService.java index b114bc470c..e80aa22e4d 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/ProcessDefinitionService.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/ProcessDefinitionService.java @@ -16,9 +16,18 @@ */ package org.apache.dolphinscheduler.api.service; +import com.alibaba.fastjson.JSON; +import com.alibaba.fastjson.JSONArray; +import com.alibaba.fastjson.JSONObject; +import com.baomidou.mybatisplus.core.metadata.IPage; +import com.baomidou.mybatisplus.extension.plugins.pagination.Page; +import com.fasterxml.jackson.core.JsonProcessingException; +import org.apache.commons.lang3.ObjectUtils; +import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.api.dto.treeview.Instance; import org.apache.dolphinscheduler.api.dto.treeview.TreeViewDto; import org.apache.dolphinscheduler.api.enums.Status; +import org.apache.dolphinscheduler.api.utils.CheckUtils; import org.apache.dolphinscheduler.api.utils.PageInfo; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.*; @@ -32,14 +41,6 @@ import org.apache.dolphinscheduler.common.utils.CollectionUtils; import org.apache.dolphinscheduler.common.utils.DateUtils; import org.apache.dolphinscheduler.common.utils.JSONUtils; import org.apache.dolphinscheduler.dao.ProcessDao; -import com.alibaba.fastjson.JSON; -import com.alibaba.fastjson.JSONArray; -import com.alibaba.fastjson.JSONObject; -import com.baomidou.mybatisplus.core.metadata.IPage; -import com.baomidou.mybatisplus.extension.plugins.pagination.Page; -import com.fasterxml.jackson.core.JsonProcessingException; -import org.apache.commons.lang3.ObjectUtils; -import org.apache.dolphinscheduler.api.utils.CheckUtils; import org.apache.dolphinscheduler.dao.entity.*; import org.apache.dolphinscheduler.dao.mapper.*; import org.slf4j.Logger; @@ -56,8 +57,10 @@ import java.io.BufferedOutputStream; import java.io.BufferedReader; import java.io.IOException; import java.io.InputStreamReader; +import java.nio.charset.StandardCharsets; import java.util.*; import java.util.concurrent.ConcurrentHashMap; +import java.util.stream.Collectors; import static org.apache.dolphinscheduler.common.Constants.CMDPARAM_SUB_PROCESS_DEFINE_ID; @@ -482,50 +485,21 @@ public class ProcessDefinitionService extends BaseDAGService { * @param response response */ public void exportProcessDefinitionById(User loginUser, String projectName, Integer processDefinitionId, HttpServletResponse response) { + //export project info Project project = projectMapper.queryByName(projectName); + //check user access for project Map checkResult = projectService.checkProjectAndAuth(loginUser, project, projectName); Status resultStatus = (Status) checkResult.get(Constants.STATUS); + if (resultStatus == Status.SUCCESS) { + //get workflow info ProcessDefinition processDefinition = processDefineMapper.queryByDefineId(processDefinitionId); - if (processDefinition != null) { - JSONObject jsonObject = JSONUtils.parseObject(processDefinition.getProcessDefinitionJson()); - JSONArray jsonArray = (JSONArray) jsonObject.get("tasks"); - for (int i = 0; i < jsonArray.size(); i++) { - JSONObject taskNode = jsonArray.getJSONObject(i); - if (taskNode.get("type") != null && taskNode.get("type") != "") { - String taskType = taskNode.getString("type"); - if(taskType.equals(TaskType.SQL.name()) || taskType.equals(TaskType.PROCEDURE.name())){ - JSONObject sqlParameters = JSONUtils.parseObject(taskNode.getString("params")); - DataSource dataSource = dataSourceMapper.selectById((Integer) sqlParameters.get("datasource")); - if (dataSource != null) { - sqlParameters.put("datasourceName", dataSource.getName()); - } - taskNode.put("params", sqlParameters); - }else if(taskType.equals(TaskType.DEPENDENT.name())){ - JSONObject dependentParameters = JSONUtils.parseObject(taskNode.getString("dependence")); - if(dependentParameters != null){ - JSONArray dependTaskList = (JSONArray) dependentParameters.get("dependTaskList"); - for (int j = 0; j < dependTaskList.size(); j++) { - JSONObject dependentTaskModel = dependTaskList.getJSONObject(j); - JSONArray dependItemList = (JSONArray) dependentTaskModel.get("dependItemList"); - for (int k = 0; k < dependItemList.size(); k++) { - JSONObject dependentItem = dependItemList.getJSONObject(k); - int definitionId = dependentItem.getInteger("definitionId"); - ProcessDefinition definition = processDefineMapper.queryByDefineId(definitionId); - if(definition != null){ - dependentItem.put("projectName",definition.getProjectName()); - dependentItem.put("definitionName",definition.getName()); - } - } - } - taskNode.put("dependence", dependentParameters); - } - } - } - } - jsonObject.put("tasks", jsonArray); - processDefinition.setProcessDefinitionJson(jsonObject.toString()); + + if (null != processDefinition) { + //correct task param which has data source or dependent param + String correctProcessDefinitionJson = addTaskNodeSpecialParam(processDefinition.getProcessDefinitionJson()); + processDefinition.setProcessDefinitionJson(correctProcessDefinitionJson); Map row = new LinkedHashMap<>(); row.put("projectName", processDefinition.getProjectName()); @@ -535,8 +509,9 @@ public class ProcessDefinitionService extends BaseDAGService { row.put("processDefinitionLocations", processDefinition.getLocations()); row.put("processDefinitionConnects", processDefinition.getConnects()); + //schedule info List schedules = scheduleMapper.queryByProcessDefinitionId(processDefinitionId); - if (schedules.size() > 0) { + if (!schedules.isEmpty()) { Schedule schedule = schedules.get(0); row.put("scheduleWarningType", schedule.getWarningType()); row.put("scheduleWarningGroupId", schedule.getWarningGroupId()); @@ -556,6 +531,8 @@ public class ProcessDefinitionService extends BaseDAGService { } } + + //create workflow json file String rowsJson = JSONUtils.toJsonString(row); response.setContentType(MediaType.APPLICATION_JSON_UTF8_VALUE); response.setHeader("Content-Disposition", "attachment;filename="+processDefinition.getName()+".json"); @@ -564,38 +541,136 @@ public class ProcessDefinitionService extends BaseDAGService { try { out = response.getOutputStream(); buff = new BufferedOutputStream(out); - buff.write(rowsJson.getBytes("UTF-8")); + buff.write(rowsJson.getBytes(StandardCharsets.UTF_8)); buff.flush(); buff.close(); } catch (IOException e) { - e.printStackTrace(); + logger.warn("export process fail", e); }finally { - try { - buff.close(); - out.close(); - } catch (Exception e) { - e.printStackTrace(); + if (null != buff) { + try { + buff.close(); + } catch (Exception e) { + logger.warn("export process buffer not close", e); + } + } + if (null != out) { + try { + out.close(); + } catch (Exception e) { + logger.warn("export process output stream not close", e); + } + } + + } + } + } + } + + /** + * correct task param which has datasource or dependent + * @param processDefinitionJson processDefinitionJson + * @return correct processDefinitionJson + */ + public String addTaskNodeSpecialParam(String processDefinitionJson) { + JSONObject jsonObject = JSONUtils.parseObject(processDefinitionJson); + JSONArray jsonArray = (JSONArray) jsonObject.get("tasks"); + + for (int i = 0; i < jsonArray.size(); i++) { + JSONObject taskNode = jsonArray.getJSONObject(i); + if (StringUtils.isNotEmpty(taskNode.getString("type"))) { + String taskType = taskNode.getString("type"); + + if(checkTaskHasDataSource(taskType)){ + // add sqlParameters + JSONObject sqlParameters = JSONUtils.parseObject(taskNode.getString("params")); + DataSource dataSource = dataSourceMapper.selectById((Integer) sqlParameters.get("datasource")); + if (null != dataSource) { + sqlParameters.put("datasourceName", dataSource.getName()); + } + taskNode.put("params", sqlParameters); + }else if(checkTaskHasDependent(taskType)){ + // add dependent param + JSONObject dependentParameters = JSONUtils.parseObject(taskNode.getString("dependence")); + + if(null != dependentParameters){ + JSONArray dependTaskList = (JSONArray) dependentParameters.get("dependTaskList"); + for (int j = 0; j < dependTaskList.size(); j++) { + JSONObject dependentTaskModel = dependTaskList.getJSONObject(j); + JSONArray dependItemList = (JSONArray) dependentTaskModel.get("dependItemList"); + for (int k = 0; k < dependItemList.size(); k++) { + JSONObject dependentItem = dependItemList.getJSONObject(k); + int definitionId = dependentItem.getInteger("definitionId"); + ProcessDefinition definition = processDefineMapper.queryByDefineId(definitionId); + if(null != definition){ + dependentItem.put("projectName",definition.getProjectName()); + dependentItem.put("definitionName",definition.getName()); + } + } + } + taskNode.put("dependence", dependentParameters); } } } } + jsonObject.put("tasks", jsonArray); + return jsonObject.toString(); + } + + /** + * check task if has dependent + * @param taskType task type + * @return if task has dependent return true else false + */ + private boolean checkTaskHasDependent(String taskType) { + return taskType.equals(TaskType.DEPENDENT.name()); + } + + /** + * check task if has data source info + * @param taskType task type + * @return if task has data source return true else false + */ + private boolean checkTaskHasDataSource(String taskType) { + return taskType.equals(TaskType.SQL.name()) || taskType.equals(TaskType.PROCEDURE.name()); } + /** + * check task if has sub process + * @param taskType task type + * @return if task has sub process return true else false + */ + private boolean checkTaskHasSubProcess(String taskType) { + return taskType.equals(TaskType.SUB_PROCESS.name()); + } + + /** + * import process definition + * @param loginUser login user + * @param file process metadata json file + * @param currentProjectName current project name + * @return + */ @Transactional(rollbackFor = Exception.class) - public Map importProcessDefinition(User loginUser, MultipartFile file) { + public Map importProcessDefinition(User loginUser, MultipartFile file, String currentProjectName) { Map result = new HashMap<>(5); - JSONObject json = null; - try(InputStreamReader inputStreamReader = new InputStreamReader( file.getInputStream(), "UTF-8" )) { + JSONObject json; + + //read workflow json + try(InputStreamReader inputStreamReader = new InputStreamReader( file.getInputStream(), StandardCharsets.UTF_8)) { BufferedReader streamReader = new BufferedReader(inputStreamReader); StringBuilder respomseStrBuilder = new StringBuilder(); - String inputStr = ""; + String inputStr; + while ((inputStr = streamReader.readLine())!= null){ respomseStrBuilder.append( inputStr ); } + json = JSONObject.parseObject( respomseStrBuilder.toString() ); - if(json != null){ - String projectName = null; + + if(null != json){ + String originProjectName = null; String processDefinitionName = null; String processDefinitionJson = null; String processDefinitionDesc = null; @@ -614,7 +689,7 @@ public class ProcessDefinitionService extends BaseDAGService { String scheduleWorkerGroupName = null; if (ObjectUtils.allNotNull(json.get("projectName"))) { - projectName = json.get("projectName").toString(); + originProjectName = json.get("projectName").toString(); } else { putMsg(result, Status.DATA_IS_NULL, "processDefinitionName"); return result; @@ -641,123 +716,226 @@ public class ProcessDefinitionService extends BaseDAGService { processDefinitionConnects = json.get("processDefinitionConnects").toString(); } - Project project = projectMapper.queryByName(projectName); - if(project != null){ - processDefinitionName = recursionProcessDefinitionName(project.getId(), processDefinitionName, 1); - } + //check user access for org project + Project originProject = projectMapper.queryByName(originProjectName); + Map checkResult = projectService.checkProjectAndAuth(loginUser, originProject, originProjectName); + Status resultStatus = (Status) checkResult.get(Constants.STATUS); - JSONObject jsonObject = JSONUtils.parseObject(processDefinitionJson); - JSONArray jsonArray = (JSONArray) jsonObject.get("tasks"); - for (int j = 0; j < jsonArray.size(); j++) { - JSONObject taskNode = jsonArray.getJSONObject(j); - String taskType = taskNode.getString("type"); - if(taskType.equals(TaskType.SQL.name()) || taskType.equals(TaskType.PROCEDURE.name())) { - JSONObject sqlParameters = JSONUtils.parseObject(taskNode.getString("params")); - List dataSources = dataSourceMapper.queryDataSourceByName(sqlParameters.getString("datasourceName")); - if (dataSources.size() > 0) { - DataSource dataSource = dataSources.get(0); - sqlParameters.put("datasource", dataSource.getId()); - } - taskNode.put("params", sqlParameters); - }else if(taskType.equals(TaskType.DEPENDENT.name())){ - JSONObject dependentParameters = JSONUtils.parseObject(taskNode.getString("dependence")); - if(dependentParameters != null){ - JSONArray dependTaskList = (JSONArray) dependentParameters.get("dependTaskList"); - for (int h = 0; h < dependTaskList.size(); h++) { - JSONObject dependentTaskModel = dependTaskList.getJSONObject(h); - JSONArray dependItemList = (JSONArray) dependentTaskModel.get("dependItemList"); - for (int k = 0; k < dependItemList.size(); k++) { - JSONObject dependentItem = dependItemList.getJSONObject(k); - Project dependentItemProject = projectMapper.queryByName(dependentItem.getString("projectName")); - if(dependentItemProject != null){ - ProcessDefinition definition = processDefineMapper.queryByDefineName(dependentItemProject.getId(),dependentItem.getString("definitionName")); - if(definition != null){ - dependentItem.put("projectId",dependentItemProject.getId()); - dependentItem.put("definitionId",definition.getId()); + if (resultStatus == Status.SUCCESS) { + //use currentProjectName to query + Project targetProject = projectMapper.queryByName(currentProjectName); + if(null != targetProject){ + processDefinitionName = recursionProcessDefinitionName(targetProject.getId(), processDefinitionName, 1); + } + + JSONObject jsonObject = JSONUtils.parseObject(processDefinitionJson); + JSONArray jsonArray = (JSONArray) jsonObject.get("tasks"); + + for (int j = 0; j < jsonArray.size(); j++) { + JSONObject taskNode = jsonArray.getJSONObject(j); + String taskType = taskNode.getString("type"); + if(checkTaskHasDataSource(taskType)) { + JSONObject sqlParameters = JSONUtils.parseObject(taskNode.getString("params")); + List dataSources = dataSourceMapper.queryDataSourceByName(sqlParameters.getString("datasourceName")); + if (!dataSources.isEmpty()) { + DataSource dataSource = dataSources.get(0); + sqlParameters.put("datasource", dataSource.getId()); + } + taskNode.put("params", sqlParameters); + }else if(checkTaskHasDependent(taskType)){ + JSONObject dependentParameters = JSONUtils.parseObject(taskNode.getString("dependence")); + if(dependentParameters != null){ + JSONArray dependTaskList = (JSONArray) dependentParameters.get("dependTaskList"); + for (int h = 0; h < dependTaskList.size(); h++) { + JSONObject dependentTaskModel = dependTaskList.getJSONObject(h); + JSONArray dependItemList = (JSONArray) dependentTaskModel.get("dependItemList"); + for (int k = 0; k < dependItemList.size(); k++) { + JSONObject dependentItem = dependItemList.getJSONObject(k); + Project dependentItemProject = projectMapper.queryByName(dependentItem.getString("projectName")); + if(dependentItemProject != null){ + ProcessDefinition definition = processDefineMapper.queryByDefineName(dependentItemProject.getId(),dependentItem.getString("definitionName")); + if(definition != null){ + dependentItem.put("projectId",dependentItemProject.getId()); + dependentItem.put("definitionId",definition.getId()); + } } } } + taskNode.put("dependence", dependentParameters); } - taskNode.put("dependence", dependentParameters); } } - } - jsonObject.put("tasks", jsonArray); - Map createProcessDefinitionResult = createProcessDefinition(loginUser,projectName,processDefinitionName,jsonObject.toString(),processDefinitionDesc,processDefinitionLocations,processDefinitionConnects); - Integer processDefinitionId = null; - if (ObjectUtils.allNotNull(createProcessDefinitionResult.get("processDefinitionId"))) { - processDefinitionId = Integer.parseInt(createProcessDefinitionResult.get("processDefinitionId").toString()); - } - if (ObjectUtils.allNotNull(json.get("scheduleCrontab")) && processDefinitionId != null) { - Date now = new Date(); - Schedule scheduleObj = new Schedule(); - scheduleObj.setProjectName(projectName); - scheduleObj.setProcessDefinitionId(processDefinitionId); - scheduleObj.setProcessDefinitionName(processDefinitionName); - scheduleObj.setCreateTime(now); - scheduleObj.setUpdateTime(now); - scheduleObj.setUserId(loginUser.getId()); - scheduleObj.setUserName(loginUser.getUserName()); - - - scheduleCrontab = json.get("scheduleCrontab").toString(); - scheduleObj.setCrontab(scheduleCrontab); - if (ObjectUtils.allNotNull(json.get("scheduleStartTime"))) { - scheduleStartTime = json.get("scheduleStartTime").toString(); - scheduleObj.setStartTime(DateUtils.stringToDate(scheduleStartTime)); - } - if (ObjectUtils.allNotNull(json.get("scheduleEndTime"))) { - scheduleEndTime = json.get("scheduleEndTime").toString(); - scheduleObj.setEndTime(DateUtils.stringToDate(scheduleEndTime)); - } - if (ObjectUtils.allNotNull(json.get("scheduleWarningType"))) { - scheduleWarningType = json.get("scheduleWarningType").toString(); - scheduleObj.setWarningType(WarningType.valueOf(scheduleWarningType)); - } - if (ObjectUtils.allNotNull(json.get("scheduleWarningGroupId"))) { - scheduleWarningGroupId = json.get("scheduleWarningGroupId").toString(); - scheduleObj.setWarningGroupId(Integer.parseInt(scheduleWarningGroupId)); - } - if (ObjectUtils.allNotNull(json.get("scheduleFailureStrategy"))) { - scheduleFailureStrategy = json.get("scheduleFailureStrategy").toString(); - scheduleObj.setFailureStrategy(FailureStrategy.valueOf(scheduleFailureStrategy)); - } - if (ObjectUtils.allNotNull(json.get("scheduleReleaseState"))) { - scheduleReleaseState = json.get("scheduleReleaseState").toString(); - scheduleObj.setReleaseState(ReleaseState.valueOf(scheduleReleaseState)); + //recursive sub-process parameter correction map key for old process id value for new process id + Map subProcessIdMap = new HashMap<>(20); + + List subProcessList = jsonArray.stream() + .filter(elem -> checkTaskHasSubProcess(JSONUtils.parseObject(elem.toString()).getString("type"))) + .collect(Collectors.toList()); + + if (!subProcessList.isEmpty()) { + importSubProcess(loginUser, targetProject, jsonArray, subProcessIdMap); } - if (ObjectUtils.allNotNull(json.get("scheduleProcessInstancePriority"))) { - scheduleProcessInstancePriority = json.get("scheduleProcessInstancePriority").toString(); - scheduleObj.setProcessInstancePriority(Priority.valueOf(scheduleProcessInstancePriority)); + + jsonObject.put("tasks", jsonArray); + + Map createProcessDefinitionResult = createProcessDefinition(loginUser,currentProjectName,processDefinitionName,jsonObject.toString(),processDefinitionDesc,processDefinitionLocations,processDefinitionConnects); + Integer processDefinitionId = null; + if (ObjectUtils.allNotNull(createProcessDefinitionResult.get("processDefinitionId"))) { + processDefinitionId = Integer.parseInt(createProcessDefinitionResult.get("processDefinitionId").toString()); } - if (ObjectUtils.allNotNull(json.get("scheduleWorkerGroupId"))) { - scheduleWorkerGroupId = json.get("scheduleWorkerGroupId").toString(); - if(scheduleWorkerGroupId != null){ - scheduleObj.setWorkerGroupId(Integer.parseInt(scheduleWorkerGroupId)); - }else{ - if (ObjectUtils.allNotNull(json.get("scheduleWorkerGroupName"))) { - scheduleWorkerGroupName = json.get("scheduleWorkerGroupName").toString(); - List workerGroups = workerGroupMapper.queryWorkerGroupByName(scheduleWorkerGroupName); - if(workerGroups.size() > 0){ - scheduleObj.setWorkerGroupId(workerGroups.get(0).getId()); + if (ObjectUtils.allNotNull(json.get("scheduleCrontab")) && processDefinitionId != null) { + Date now = new Date(); + Schedule scheduleObj = new Schedule(); + scheduleObj.setProjectName(currentProjectName); + scheduleObj.setProcessDefinitionId(processDefinitionId); + scheduleObj.setProcessDefinitionName(processDefinitionName); + scheduleObj.setCreateTime(now); + scheduleObj.setUpdateTime(now); + scheduleObj.setUserId(loginUser.getId()); + scheduleObj.setUserName(loginUser.getUserName()); + + + scheduleCrontab = json.get("scheduleCrontab").toString(); + scheduleObj.setCrontab(scheduleCrontab); + if (ObjectUtils.allNotNull(json.get("scheduleStartTime"))) { + scheduleStartTime = json.get("scheduleStartTime").toString(); + scheduleObj.setStartTime(DateUtils.stringToDate(scheduleStartTime)); + } + if (ObjectUtils.allNotNull(json.get("scheduleEndTime"))) { + scheduleEndTime = json.get("scheduleEndTime").toString(); + scheduleObj.setEndTime(DateUtils.stringToDate(scheduleEndTime)); + } + if (ObjectUtils.allNotNull(json.get("scheduleWarningType"))) { + scheduleWarningType = json.get("scheduleWarningType").toString(); + scheduleObj.setWarningType(WarningType.valueOf(scheduleWarningType)); + } + if (ObjectUtils.allNotNull(json.get("scheduleWarningGroupId"))) { + scheduleWarningGroupId = json.get("scheduleWarningGroupId").toString(); + scheduleObj.setWarningGroupId(Integer.parseInt(scheduleWarningGroupId)); + } + if (ObjectUtils.allNotNull(json.get("scheduleFailureStrategy"))) { + scheduleFailureStrategy = json.get("scheduleFailureStrategy").toString(); + scheduleObj.setFailureStrategy(FailureStrategy.valueOf(scheduleFailureStrategy)); + } + if (ObjectUtils.allNotNull(json.get("scheduleReleaseState"))) { + scheduleReleaseState = json.get("scheduleReleaseState").toString(); + scheduleObj.setReleaseState(ReleaseState.valueOf(scheduleReleaseState)); + } + if (ObjectUtils.allNotNull(json.get("scheduleProcessInstancePriority"))) { + scheduleProcessInstancePriority = json.get("scheduleProcessInstancePriority").toString(); + scheduleObj.setProcessInstancePriority(Priority.valueOf(scheduleProcessInstancePriority)); + } + if (ObjectUtils.allNotNull(json.get("scheduleWorkerGroupId"))) { + scheduleWorkerGroupId = json.get("scheduleWorkerGroupId").toString(); + if(scheduleWorkerGroupId != null){ + scheduleObj.setWorkerGroupId(Integer.parseInt(scheduleWorkerGroupId)); + }else{ + if (ObjectUtils.allNotNull(json.get("scheduleWorkerGroupName"))) { + scheduleWorkerGroupName = json.get("scheduleWorkerGroupName").toString(); + List workerGroups = workerGroupMapper.queryWorkerGroupByName(scheduleWorkerGroupName); + if(!workerGroups.isEmpty()){ + scheduleObj.setWorkerGroupId(workerGroups.get(0).getId()); + } } } } + scheduleMapper.insert(scheduleObj); } - scheduleMapper.insert(scheduleObj); + + putMsg(result, Status.SUCCESS); + return result; } }else{ - putMsg(result, Status.EXPORT_PROCESS_DEFINE_BY_ID_ERROR); + putMsg(result, Status.IMPORT_PROCESS_DEFINE_ERROR); return result; } } catch (IOException e) { throw new RuntimeException(e.getMessage(), e); } - putMsg(result, Status.SUCCESS); return result; } + /** + * check import process has sub process + * recursion create sub process + * @param loginUser login user + * @param targetProject target project + */ + public void importSubProcess(User loginUser, Project targetProject, JSONArray jsonArray, Map subProcessIdMap) { + for (int i = 0; i < jsonArray.size(); i++) { + JSONObject taskNode = jsonArray.getJSONObject(i); + String taskType = taskNode.getString("type"); + + if (checkTaskHasSubProcess(taskType)) { + //get sub process info + JSONObject subParams = JSONUtils.parseObject(taskNode.getString("params")); + Integer subProcessId = subParams.getInteger("processDefinitionId"); + ProcessDefinition subProcess = processDefineMapper.queryByDefineId(subProcessId); + String subProcessJson = subProcess.getProcessDefinitionJson(); + //check current project has sub process + ProcessDefinition currentProjectSubProcess = processDefineMapper.queryByDefineName(targetProject.getId(), subProcess.getName()); + + if (null == currentProjectSubProcess) { + JSONArray subJsonArray = (JSONArray) JSONUtils.parseObject(subProcess.getProcessDefinitionJson()).get("tasks"); + + List subProcessList = subJsonArray.stream() + .filter(item -> checkTaskHasSubProcess(JSONUtils.parseObject(item.toString()).getString("type"))) + .collect(Collectors.toList()); + + if (!subProcessList.isEmpty()) { + importSubProcess(loginUser, targetProject, subJsonArray, subProcessIdMap); + //sub process processId correct + if (!subProcessIdMap.isEmpty()) { + + for (Map.Entry entry : subProcessIdMap.entrySet()) { + String oldSubProcessId = "\"processDefinitionId\":" + entry.getKey(); + String newSubProcessId = "\"processDefinitionId\":" + entry.getValue(); + subProcessJson = subProcessJson.replaceAll(oldSubProcessId, newSubProcessId); + } + + subProcessIdMap.clear(); + } + } + + //if sub-process recursion + Date now = new Date(); + //create sub process in target project + ProcessDefinition processDefine = new ProcessDefinition(); + processDefine.setName(subProcess.getName()); + processDefine.setVersion(subProcess.getVersion()); + processDefine.setReleaseState(subProcess.getReleaseState()); + processDefine.setProjectId(targetProject.getId()); + processDefine.setUserId(loginUser.getId()); + processDefine.setProcessDefinitionJson(subProcessJson); + processDefine.setDescription(subProcess.getDescription()); + processDefine.setLocations(subProcess.getLocations()); + processDefine.setConnects(subProcess.getConnects()); + processDefine.setTimeout(subProcess.getTimeout()); + processDefine.setTenantId(subProcess.getTenantId()); + processDefine.setGlobalParams(subProcess.getGlobalParams()); + processDefine.setCreateTime(now); + processDefine.setUpdateTime(now); + processDefine.setFlag(subProcess.getFlag()); + processDefine.setReceivers(subProcess.getReceivers()); + processDefine.setReceiversCc(subProcess.getReceiversCc()); + processDefineMapper.insert(processDefine); + + logger.info("create sub process, project: {}, process name: {}", targetProject.getName(), processDefine.getName()); + + //modify task node + ProcessDefinition newSubProcessDefine = processDefineMapper.queryByDefineName(processDefine.getProjectId(),processDefine.getName()); + + if (null != newSubProcessDefine) { + subProcessIdMap.put(subProcessId, newSubProcessDefine.getId()); + subParams.put("processDefinitionId", newSubProcessDefine.getId()); + taskNode.put("params", subParams); + } + + } + } + } + } /** diff --git a/dolphinscheduler-api/src/test/java/org/apache/dolphinscheduler/api/service/ProcessDefinitionServiceTest.java b/dolphinscheduler-api/src/test/java/org/apache/dolphinscheduler/api/service/ProcessDefinitionServiceTest.java index 90c800fcaa..b8207972f6 100644 --- a/dolphinscheduler-api/src/test/java/org/apache/dolphinscheduler/api/service/ProcessDefinitionServiceTest.java +++ b/dolphinscheduler-api/src/test/java/org/apache/dolphinscheduler/api/service/ProcessDefinitionServiceTest.java @@ -16,38 +16,80 @@ */ package org.apache.dolphinscheduler.api.service; +import com.alibaba.fastjson.JSON; +import com.alibaba.fastjson.JSONArray; +import com.alibaba.fastjson.JSONObject; import org.apache.dolphinscheduler.api.ApiApplicationServer; import org.apache.dolphinscheduler.api.enums.Status; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.UserType; +import org.apache.dolphinscheduler.common.utils.FileUtils; +import org.apache.dolphinscheduler.common.utils.JSONUtils; +import org.apache.dolphinscheduler.dao.entity.DataSource; +import org.apache.dolphinscheduler.dao.entity.ProcessDefinition; +import org.apache.dolphinscheduler.dao.entity.Project; import org.apache.dolphinscheduler.dao.entity.User; -import com.alibaba.fastjson.JSON; +import org.apache.dolphinscheduler.dao.mapper.DataSourceMapper; +import org.apache.dolphinscheduler.dao.mapper.ProcessDefinitionMapper; +import org.apache.dolphinscheduler.dao.mapper.ProjectMapper; +import org.apache.http.entity.ContentType; +import org.json.JSONException; import org.junit.Assert; import org.junit.Test; import org.junit.runner.RunWith; +import org.mockito.InjectMocks; +import org.mockito.Mock; +import org.mockito.Mockito; +import org.mockito.junit.MockitoJUnitRunner; +import org.skyscreamer.jsonassert.JSONAssert; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import org.springframework.beans.factory.annotation.Autowired; import org.springframework.boot.test.context.SpringBootTest; -import org.springframework.test.context.junit4.SpringRunner; +import org.springframework.mock.web.MockMultipartFile; +import org.springframework.web.multipart.MultipartFile; +import java.io.File; +import java.io.FileInputStream; +import java.io.IOException; +import java.text.MessageFormat; +import java.util.HashMap; import java.util.Map; -@RunWith(SpringRunner.class) +@RunWith(MockitoJUnitRunner.Silent.class) @SpringBootTest(classes = ApiApplicationServer.class) public class ProcessDefinitionServiceTest { private static final Logger logger = LoggerFactory.getLogger(ProcessDefinitionServiceTest.class); - @Autowired + @InjectMocks ProcessDefinitionService processDefinitionService; + @Mock + private DataSourceMapper dataSourceMapper; + + @Mock + private ProcessDefinitionMapper processDefineMapper; + + @Mock + private ProjectMapper projectMapper; + + @Mock + private ProjectService projectService; + @Test public void queryProccessDefinitionList() throws Exception { + String projectName = "project_test1"; + Mockito.when(projectMapper.queryByName(projectName)).thenReturn(getProject(projectName)); + Project project = getProject(projectName); User loginUser = new User(); loginUser.setId(-1); loginUser.setUserType(UserType.GENERAL_USER); + Map result = new HashMap<>(5); + putMsg(result, Status.PROJECT_NOT_FOUNT, projectName); + + Mockito.when(projectService.checkProjectAndAuth(loginUser,project,projectName)).thenReturn(result); + Map map = processDefinitionService.queryProccessDefinitionList(loginUser,"project_test1"); Assert.assertEquals(Status.PROJECT_NOT_FOUNT, map.get(Constants.STATUS)); logger.info(JSON.toJSONString(map)); @@ -55,10 +97,20 @@ public class ProcessDefinitionServiceTest { @Test public void queryProcessDefinitionListPagingTest() throws Exception { + String projectName = "project_test1"; + Mockito.when(projectMapper.queryByName(projectName)).thenReturn(getProject(projectName)); + + Project project = getProject(projectName); User loginUser = new User(); loginUser.setId(-1); loginUser.setUserType(UserType.GENERAL_USER); + + Map result = new HashMap<>(5); + putMsg(result, Status.PROJECT_NOT_FOUNT, projectName); + + Mockito.when(projectService.checkProjectAndAuth(loginUser,project,projectName)).thenReturn(result); + Map map = processDefinitionService.queryProcessDefinitionListPaging(loginUser, "project_test1", "",1, 5,0); Assert.assertEquals(Status.PROJECT_NOT_FOUNT, map.get(Constants.STATUS)); @@ -67,13 +119,243 @@ public class ProcessDefinitionServiceTest { @Test public void deleteProcessDefinitionByIdTest() throws Exception { + String projectName = "project_test1"; + Mockito.when(projectMapper.queryByName(projectName)).thenReturn(getProject(projectName)); + Project project = getProject(projectName); User loginUser = new User(); loginUser.setId(-1); loginUser.setUserType(UserType.GENERAL_USER); - Map map = processDefinitionService.deleteProcessDefinitionById(loginUser, "li_sql_test", 6); + + Map result = new HashMap<>(5); + putMsg(result, Status.PROJECT_NOT_FOUNT, projectName); + Mockito.when(projectService.checkProjectAndAuth(loginUser,project,projectName)).thenReturn(result); + + Map map = processDefinitionService.deleteProcessDefinitionById(loginUser, "project_test1", 6); Assert.assertEquals(Status.PROJECT_NOT_FOUNT, map.get(Constants.STATUS)); logger.info(JSON.toJSONString(map)); } + + /** + * add datasource param and dependent when export process + * @throws JSONException + */ + @Test + public void testAddTaskNodeSpecialParam() throws JSONException { + + Mockito.when(dataSourceMapper.selectById(1)).thenReturn(getDataSource()); + Mockito.when(processDefineMapper.queryByDefineId(2)).thenReturn(getProcessDefinition()); + + + String sqlDependentJson = "{\"globalParams\":[]," + + "\"tasks\":[{\"type\":\"SQL\",\"id\":\"tasks-27297\",\"name\":\"sql\"," + + "\"params\":{\"type\":\"MYSQL\",\"datasource\":1,\"sql\":\"select * from test\"," + + "\"udfs\":\"\",\"sqlType\":\"1\",\"title\":\"\",\"receivers\":\"\",\"receiversCc\":\"\",\"showType\":\"TABLE\"" + + ",\"localParams\":[],\"connParams\":\"\"," + + "\"preStatements\":[],\"postStatements\":[]}," + + "\"description\":\"\",\"runFlag\":\"NORMAL\",\"dependence\":{},\"maxRetryTimes\":\"0\"," + + "\"retryInterval\":\"1\",\"timeout\":{\"strategy\":\"\"," + + "\"enable\":false},\"taskInstancePriority\":\"MEDIUM\",\"workerGroupId\":-1," + + "\"preTasks\":[\"dependent\"]},{\"type\":\"DEPENDENT\",\"id\":\"tasks-33787\"," + + "\"name\":\"dependent\",\"params\":{},\"description\":\"\",\"runFlag\":\"NORMAL\"," + + "\"dependence\":{\"relation\":\"AND\",\"dependTaskList\":[{\"relation\":\"AND\"," + + "\"dependItemList\":[{\"projectId\":2,\"definitionId\":46,\"depTasks\":\"ALL\"," + + "\"cycle\":\"day\",\"dateValue\":\"today\"}]}]},\"maxRetryTimes\":\"0\",\"retryInterval\":\"1\"," + + "\"timeout\":{\"strategy\":\"\",\"enable\":false},\"taskInstancePriority\":\"MEDIUM\"," + + "\"workerGroupId\":-1,\"preTasks\":[]}],\"tenantId\":1,\"timeout\":0}"; + + String corSqlDependentJson = processDefinitionService.addTaskNodeSpecialParam(sqlDependentJson); + + + JSONAssert.assertEquals(sqlDependentJson,corSqlDependentJson,false); + + } + + /** + * import sub process test + */ + @Test + public void testImportSubProcess() { + + User loginUser = new User(); + loginUser.setId(1); + loginUser.setUserType(UserType.ADMIN_USER); + + Project testProject = getProject("test"); + + //Recursive subprocess sub2 process in sub1 process and sub1process in top process + String topProcessJson = "{\"globalParams\":[]," + + "\"tasks\":[{\"type\":\"SHELL\",\"id\":\"tasks-38634\",\"name\":\"shell1\"," + + "\"params\":{\"resourceList\":[],\"localParams\":[],\"rawScript\":\"#!/bin/bash\\necho \\\"shell-1\\\"\"}," + + "\"description\":\"\",\"runFlag\":\"NORMAL\",\"dependence\":{},\"maxRetryTimes\":\"0\"," + + "\"retryInterval\":\"1\",\"timeout\":{\"strategy\":\"\",\"interval\":null,\"enable\":false}," + + "\"taskInstancePriority\":\"MEDIUM\",\"workerGroupId\":-1,\"preTasks\":[]}," + + "{\"type\":\"SUB_PROCESS\",\"id\":\"tasks-44207\",\"name\":\"shell-4\"," + + "\"params\":{\"processDefinitionId\":39},\"description\":\"\",\"runFlag\":\"NORMAL\"," + + "\"dependence\":{},\"timeout\":{\"strategy\":\"\",\"interval\":null,\"enable\":false}," + + "\"taskInstancePriority\":\"MEDIUM\",\"workerGroupId\":-1," + + "\"preTasks\":[\"shell1\"]}],\"tenantId\":1,\"timeout\":0}"; + + String sub1ProcessJson = "{\"globalParams\":[],\"tasks\":[{\"type\":\"SHELL\",\"id\":\"tasks-84090\"," + + "\"name\":\"shell-4\",\"params\":{\"resourceList\":[],\"localParams\":[],\"rawScript\":\"#!/bin/bash\\necho \\\"shell-4\\\"\"}," + + "\"description\":\"\",\"runFlag\":\"NORMAL\",\"dependence\":{},\"maxRetryTimes\":\"0\"," + + "\"retryInterval\":\"1\",\"timeout\":{\"strategy\":\"\",\"interval\":null,\"enable\":false}," + + "\"taskInstancePriority\":\"MEDIUM\",\"workerGroupId\":-1,\"preTasks\":[]},{\"type\":\"SUB_PROCESS\"," + + "\"id\":\"tasks-87364\",\"name\":\"shell-5\"," + + "\"params\":{\"processDefinitionId\":46},\"description\":\"\",\"runFlag\":\"NORMAL\",\"dependence\":{}," + + "\"timeout\":{\"strategy\":\"\",\"interval\":null,\"enable\":false},\"taskInstancePriority\":\"MEDIUM\"," + + "\"workerGroupId\":-1,\"preTasks\":[\"shell-4\"]}],\"tenantId\":1,\"timeout\":0}"; + + String sub2ProcessJson = "{\"globalParams\":[]," + + "\"tasks\":[{\"type\":\"SHELL\",\"id\":\"tasks-52423\",\"name\":\"shell-5\"," + + "\"params\":{\"resourceList\":[],\"localParams\":[],\"rawScript\":\"echo \\\"shell-5\\\"\"},\"description\":\"\"," + + "\"runFlag\":\"NORMAL\",\"dependence\":{},\"maxRetryTimes\":\"0\",\"retryInterval\":\"1\"," + + "\"timeout\":{\"strategy\":\"\",\"interval\":null,\"enable\":false},\"taskInstancePriority\":\"MEDIUM\",\"workerGroupId\":-1," + + "\"preTasks\":[]}],\"tenantId\":1,\"timeout\":0}"; + + + JSONObject jsonObject = JSONUtils.parseObject(topProcessJson); + JSONArray jsonArray = (JSONArray) jsonObject.get("tasks"); + + String originSubJson = jsonArray.toString(); + + Map subProcessIdMap = new HashMap<>(20); + + ProcessDefinition shellDefinition1 = new ProcessDefinition(); + shellDefinition1.setId(39); + shellDefinition1.setName("shell-4"); + shellDefinition1.setProjectId(2); + shellDefinition1.setProcessDefinitionJson(sub1ProcessJson); + + ProcessDefinition shellDefinition2 = new ProcessDefinition(); + shellDefinition2.setId(46); + shellDefinition2.setName("shell-5"); + shellDefinition2.setProjectId(2); + shellDefinition2.setProcessDefinitionJson(sub2ProcessJson); + + Mockito.when(processDefineMapper.queryByDefineId(39)).thenReturn(shellDefinition1); + Mockito.when(processDefineMapper.queryByDefineId(46)).thenReturn(shellDefinition2); + Mockito.when(processDefineMapper.queryByDefineName(testProject.getId(), "shell-5")).thenReturn(null); + Mockito.when(processDefineMapper.queryByDefineName(testProject.getId(), "shell-4")).thenReturn(null); + Mockito.when(processDefineMapper.queryByDefineName(testProject.getId(), "testProject")).thenReturn(shellDefinition2); + + processDefinitionService.importSubProcess(loginUser,testProject,jsonArray,subProcessIdMap); + + String correctSubJson = jsonArray.toString(); + + Assert.assertEquals(originSubJson, correctSubJson); + + } + + @Test + public void testImportProcessDefinitionById() throws IOException { + + String processJson = "{\"projectName\":\"testProject\",\"processDefinitionName\":\"shell-4\"," + + "\"processDefinitionJson\":\"{\\\"tenantId\\\":1,\\\"globalParams\\\":[]," + + "\\\"tasks\\\":[{\\\"workerGroupId\\\":-1,\\\"description\\\":\\\"\\\",\\\"runFlag\\\":\\\"NORMAL\\\"," + + "\\\"type\\\":\\\"SHELL\\\",\\\"params\\\":{\\\"rawScript\\\":\\\"#!/bin/bash\\\\necho \\\\\\\"shell-4\\\\\\\"\\\"," + + "\\\"localParams\\\":[],\\\"resourceList\\\":[]},\\\"timeout\\\":{\\\"enable\\\":false,\\\"strategy\\\":\\\"\\\"}," + + "\\\"maxRetryTimes\\\":\\\"0\\\",\\\"taskInstancePriority\\\":\\\"MEDIUM\\\",\\\"name\\\":\\\"shell-4\\\"," + + "\\\"dependence\\\":{},\\\"retryInterval\\\":\\\"1\\\",\\\"preTasks\\\":[],\\\"id\\\":\\\"tasks-84090\\\"}," + + "{\\\"taskInstancePriority\\\":\\\"MEDIUM\\\",\\\"name\\\":\\\"shell-5\\\",\\\"workerGroupId\\\":-1," + + "\\\"description\\\":\\\"\\\",\\\"dependence\\\":{},\\\"preTasks\\\":[\\\"shell-4\\\"],\\\"id\\\":\\\"tasks-87364\\\"," + + "\\\"runFlag\\\":\\\"NORMAL\\\",\\\"type\\\":\\\"SUB_PROCESS\\\",\\\"params\\\":{\\\"processDefinitionId\\\":46}," + + "\\\"timeout\\\":{\\\"enable\\\":false,\\\"strategy\\\":\\\"\\\"}}],\\\"timeout\\\":0}\"," + + "\"processDefinitionDescription\":\"\",\"processDefinitionLocations\":\"{\\\"tasks-84090\\\":{\\\"name\\\":\\\"shell-4\\\"," + + "\\\"targetarr\\\":\\\"\\\",\\\"x\\\":128,\\\"y\\\":114},\\\"tasks-87364\\\":{\\\"name\\\":\\\"shell-5\\\"," + + "\\\"targetarr\\\":\\\"tasks-84090\\\",\\\"x\\\":266,\\\"y\\\":115}}\"," + + "\"processDefinitionConnects\":\"[{\\\"endPointSourceId\\\":\\\"tasks-84090\\\"," + + "\\\"endPointTargetId\\\":\\\"tasks-87364\\\"}]\"}"; + + String subProcessJson = "{\"globalParams\":[]," + + "\"tasks\":[{\"type\":\"SHELL\",\"id\":\"tasks-52423\",\"name\":\"shell-5\"," + + "\"params\":{\"resourceList\":[],\"localParams\":[],\"rawScript\":\"echo \\\"shell-5\\\"\"},\"description\":\"\"," + + "\"runFlag\":\"NORMAL\",\"dependence\":{},\"maxRetryTimes\":\"0\",\"retryInterval\":\"1\"," + + "\"timeout\":{\"strategy\":\"\",\"interval\":null,\"enable\":false},\"taskInstancePriority\":\"MEDIUM\",\"workerGroupId\":-1," + + "\"preTasks\":[]}],\"tenantId\":1,\"timeout\":0}"; + + FileUtils.writeStringToFile(new File("/tmp/task.json"),processJson); + + File file = new File("/tmp/task.json"); + + FileInputStream fileInputStream = new FileInputStream("/tmp/task.json"); + + MultipartFile multipartFile = new MockMultipartFile(file.getName(), file.getName(), + ContentType.APPLICATION_OCTET_STREAM.toString(), fileInputStream); + + User loginUser = new User(); + loginUser.setId(1); + loginUser.setUserType(UserType.ADMIN_USER); + + String currentProjectName = "testProject"; + Map result = new HashMap<>(5); + putMsg(result, Status.SUCCESS, currentProjectName); + + ProcessDefinition shellDefinition2 = new ProcessDefinition(); + shellDefinition2.setId(46); + shellDefinition2.setName("shell-5"); + shellDefinition2.setProjectId(2); + shellDefinition2.setProcessDefinitionJson(subProcessJson); + + Mockito.when(projectMapper.queryByName(currentProjectName)).thenReturn(getProject(currentProjectName)); + Mockito.when(projectService.checkProjectAndAuth(loginUser, getProject(currentProjectName), currentProjectName)).thenReturn(result); + Mockito.when(processDefineMapper.queryByDefineId(46)).thenReturn(shellDefinition2); + + //import process + Map importProcessResult = processDefinitionService.importProcessDefinition(loginUser, multipartFile, currentProjectName); + + Assert.assertEquals(Status.SUCCESS, importProcessResult.get(Constants.STATUS)); + + boolean delete = file.delete(); + + Assert.assertTrue(delete); + + + } + + /** + * get mock datasource + * @return DataSource + */ + private DataSource getDataSource(){ + DataSource dataSource = new DataSource(); + dataSource.setId(2); + dataSource.setName("test"); + return dataSource; + } + + /** + * get mock processDefinition + * @return ProcessDefinition + */ + private ProcessDefinition getProcessDefinition(){ + ProcessDefinition processDefinition = new ProcessDefinition(); + processDefinition.setId(46); + processDefinition.setName("testProject"); + processDefinition.setProjectId(2); + return processDefinition; + } + + /** + * get mock Project + * @param projectName projectName + * @return Project + */ + private Project getProject(String projectName){ + Project project = new Project(); + project.setId(1); + project.setName(projectName); + project.setUserId(1); + return project; + } + + private void putMsg(Map result, Status status, Object... statusParams) { + result.put(Constants.STATUS, status); + if (statusParams != null && statusParams.length > 0) { + result.put(Constants.MSG, MessageFormat.format(status.getMsg(), statusParams)); + } else { + result.put(Constants.MSG, status.getMsg()); + } + } } \ No newline at end of file diff --git a/dolphinscheduler-ui/src/js/module/components/fileUpdate/definitionUpdate.vue b/dolphinscheduler-ui/src/js/module/components/fileUpdate/definitionUpdate.vue index e8d8ee3087..95851b7d11 100644 --- a/dolphinscheduler-ui/src/js/module/components/fileUpdate/definitionUpdate.vue +++ b/dolphinscheduler-ui/src/js/module/components/fileUpdate/definitionUpdate.vue @@ -141,6 +141,7 @@ let self = this let formData = new FormData() formData.append('file', this.file) + formData.append('projectName',this.store.state.dag.projectName) io.post(`projects/import-definition`, res => { this.$message.success(res.msg) resolve() diff --git a/pom.xml b/pom.xml index 6892b387f1..cebff6479f 100644 --- a/pom.xml +++ b/pom.xml @@ -675,6 +675,7 @@ **/api/service/WorkerGroupServiceTest.java **/api/service/AlertGroupServiceTest.java **/api/service/ProjectServiceTest.java + **/api/service/ProcessDefinitionServiceTest.java **/api/service/UdfFuncServiceTest.java **/alert/utils/ExcelUtilsTest.java **/alert/utils/FuncUtilsTest.java From 65bb0e03b77d041ccd5f8a09352e02c4fe6d0896 Mon Sep 17 00:00:00 2001 From: loushang Date: Thu, 2 Jan 2020 20:08:14 +0800 Subject: [PATCH 05/40] fix SessionServiceTest testSignOut bug (add mock) --- .../dolphinscheduler/api/service/SessionServiceTest.java | 3 +++ 1 file changed, 3 insertions(+) diff --git a/dolphinscheduler-api/src/test/java/org/apache/dolphinscheduler/api/service/SessionServiceTest.java b/dolphinscheduler-api/src/test/java/org/apache/dolphinscheduler/api/service/SessionServiceTest.java index 72545a347c..ce334686f6 100644 --- a/dolphinscheduler-api/src/test/java/org/apache/dolphinscheduler/api/service/SessionServiceTest.java +++ b/dolphinscheduler-api/src/test/java/org/apache/dolphinscheduler/api/service/SessionServiceTest.java @@ -121,6 +121,9 @@ public class SessionServiceTest { String ip = "127.0.0.1"; User user = new User(); user.setId(userId); + + Mockito.when(sessionMapper.queryByUserIdAndIp(userId,ip)).thenReturn(getSession()); + sessionService.signOut(ip ,user); } From b26dcba685afdfb792efca5ce5f96d3d5bef7548 Mon Sep 17 00:00:00 2001 From: AmoryWang <2421902248@qq.com> Date: Fri, 3 Jan 2020 10:09:37 +0800 Subject: [PATCH 07/40] Feature ut common thread (#1674) * add common thread ut * Update ThreadUtilsTest.java add license header * delete useless file Co-authored-by: dailidong --- pom.xml | 1 + 1 file changed, 1 insertion(+) diff --git a/pom.xml b/pom.xml index cebff6479f..a0f5bcba6f 100644 --- a/pom.xml +++ b/pom.xml @@ -661,6 +661,7 @@ **/common/utils/*.java + **/common/threadutils/*.java **/common/graph/*.java **/common/queue/*.java **/api/utils/CheckUtilsTest.java From a9cd31eca6cafe4ef2cb6c563764ad276f17d3e4 Mon Sep 17 00:00:00 2001 From: Tboy Date: Fri, 3 Jan 2020 11:14:19 +0800 Subject: [PATCH 08/40] remove commons-lang3 and commons-lang dependancy for Dao module (#1663) * remove commons-lang3 dependancy * remove commons-lang3 and commons-lang dependancy --- .../common/utils/ArrayUtils.java | 26 +++++++++++++++++++ dolphinscheduler-dao/pom.xml | 8 ------ .../apache/dolphinscheduler/dao/AlertDao.java | 2 +- .../dolphinscheduler/dao/ProcessDao.java | 5 ++-- .../dolphinscheduler/dao/TaskRecordDao.java | 2 +- .../dao/upgrade/UpgradeDao.java | 3 +-- .../dolphinscheduler/dao/utils/DagHelper.java | 2 +- 7 files changed, 32 insertions(+), 16 deletions(-) create mode 100644 dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/ArrayUtils.java diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/ArrayUtils.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/ArrayUtils.java new file mode 100644 index 0000000000..163c56bbcc --- /dev/null +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/ArrayUtils.java @@ -0,0 +1,26 @@ +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.dolphinscheduler.common.utils; + + +public class ArrayUtils { + + public static boolean isEmpty(final int[] array) { + return array == null || array.length == 0; + } +} diff --git a/dolphinscheduler-dao/pom.xml b/dolphinscheduler-dao/pom.xml index af1cdb8b78..c994716517 100644 --- a/dolphinscheduler-dao/pom.xml +++ b/dolphinscheduler-dao/pom.xml @@ -108,14 +108,6 @@ commons-httpclient commons-httpclient - - org.apache.commons - commons-lang3 - - - commons-lang - commons-lang - com.alibaba diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/AlertDao.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/AlertDao.java index 2f2314c91f..19dbf46a6b 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/AlertDao.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/AlertDao.java @@ -17,9 +17,9 @@ package org.apache.dolphinscheduler.dao; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.springframework.beans.factory.annotation.Autowired; import org.springframework.stereotype.Component; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.enums.AlertStatus; import org.apache.dolphinscheduler.common.enums.AlertType; import org.apache.dolphinscheduler.common.enums.ShowType; diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/ProcessDao.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/ProcessDao.java index 66948951f6..95e18cc9a9 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/ProcessDao.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/ProcessDao.java @@ -18,20 +18,19 @@ package org.apache.dolphinscheduler.dao; import com.alibaba.fastjson.JSONObject; import com.cronutils.model.Cron; -import org.apache.commons.lang3.ArrayUtils; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.*; import org.apache.dolphinscheduler.common.model.DateInterval; import org.apache.dolphinscheduler.common.model.TaskNode; import org.apache.dolphinscheduler.common.process.Property; import org.apache.dolphinscheduler.common.queue.ITaskQueue; -import org.apache.dolphinscheduler.common.queue.TaskQueueFactory; import org.apache.dolphinscheduler.common.task.subprocess.SubProcessParameters; +import org.apache.dolphinscheduler.common.utils.ArrayUtils; import org.apache.dolphinscheduler.common.utils.DateUtils; import org.apache.dolphinscheduler.common.utils.IpUtils; import org.apache.dolphinscheduler.common.utils.JSONUtils; import org.apache.dolphinscheduler.common.utils.ParameterUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.entity.*; import org.apache.dolphinscheduler.dao.mapper.*; import org.apache.dolphinscheduler.dao.utils.cron.CronUtils; diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/TaskRecordDao.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/TaskRecordDao.java index 4e6b1df7e6..f7ba1054a5 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/TaskRecordDao.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/TaskRecordDao.java @@ -20,11 +20,11 @@ import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.TaskRecordStatus; import org.apache.dolphinscheduler.common.utils.CollectionUtils; import org.apache.dolphinscheduler.common.utils.DateUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.entity.TaskRecord; import org.apache.commons.configuration.Configuration; import org.apache.commons.configuration.ConfigurationException; import org.apache.commons.configuration.PropertiesConfiguration; -import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/upgrade/UpgradeDao.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/upgrade/UpgradeDao.java index 1dc4ad14e7..96338248b8 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/upgrade/UpgradeDao.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/upgrade/UpgradeDao.java @@ -17,14 +17,13 @@ package org.apache.dolphinscheduler.dao.upgrade; import com.alibaba.druid.pool.DruidDataSource; -import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.DbType; import org.apache.dolphinscheduler.common.utils.ConnectionUtils; import org.apache.dolphinscheduler.common.utils.SchemaUtils; import org.apache.dolphinscheduler.common.utils.ScriptRunner; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.AbstractBaseDao; import org.apache.dolphinscheduler.dao.datasource.ConnectionFactory; -import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/utils/DagHelper.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/utils/DagHelper.java index c4b312d608..4096442f2e 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/utils/DagHelper.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/utils/DagHelper.java @@ -24,9 +24,9 @@ import org.apache.dolphinscheduler.common.model.TaskNodeRelation; import org.apache.dolphinscheduler.common.process.ProcessDag; import org.apache.dolphinscheduler.common.utils.CollectionUtils; import org.apache.dolphinscheduler.common.utils.JSONUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.entity.ProcessData; import org.apache.dolphinscheduler.dao.entity.TaskInstance; -import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; From a80e90276f8e914516b4c882c7ad5bfe100d81f4 Mon Sep 17 00:00:00 2001 From: Yelli <51317527+Yeleights@users.noreply.github.com> Date: Fri, 3 Jan 2020 11:24:01 +0800 Subject: [PATCH 09/40] fix sonar bug: not enough arguments (#1687) * modify FileUtils.readFile2Str * #1300 Add right alignment function in sql email content * cancel formatted for alert_mail_template.ftl * #747 sql task password Log desensitization * cancel mail_temple * edit ExcelUtils * modify test method name * #747 sql task password Log desensitization * #1544 workflow import * Constants add DATASOURCE_PASSWORD_REGEX * #747 sql task password Log desensitization * deal with import project have sub process * modify export process addTaskNodeParam method name * add testAddTaskNodeSpecialParam UT * add ProcessDefinitionServiceTest-ut to pom * add testImportSubProcess in ProcessDefinitionServiceTest * add testImportSubProcess in ProcessDefinitionServiceTest * add testImportProcessDefinition * fix sonar bug: not enough arguments --- .../api/controller/DataAnalysisController.java | 2 +- .../api/controller/DataSourceController.java | 2 +- .../dolphinscheduler/api/controller/TenantController.java | 2 +- .../dolphinscheduler/api/controller/UsersController.java | 6 +++--- .../dolphinscheduler/api/service/SchedulerService.java | 4 ++-- .../apache/dolphinscheduler/common/utils/HadoopUtils.java | 2 +- .../org/apache/dolphinscheduler/common/utils/HttpUtils.java | 2 +- 7 files changed, 10 insertions(+), 10 deletions(-) diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/DataAnalysisController.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/DataAnalysisController.java index 92897ac7ff..1b1dc65e01 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/DataAnalysisController.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/DataAnalysisController.java @@ -188,7 +188,7 @@ public class DataAnalysisController extends BaseController{ public Result countQueueState(@ApiIgnore @RequestAttribute(value = Constants.SESSION_USER) User loginUser, @RequestParam(value="projectId", required=false, defaultValue = "0") int projectId){ try{ - logger.info("count command state, user:{}, start date: {}, end date:{}, project id {}", + logger.info("count command state, user:{}, project id {}", loginUser.getUserName(), projectId); Map result = dataAnalysisService.countQueueState(loginUser, projectId); return returnDataList(result); diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/DataSourceController.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/DataSourceController.java index d4844a693d..9c04bf7d5c 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/DataSourceController.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/DataSourceController.java @@ -461,7 +461,7 @@ public class DataSourceController extends BaseController { @GetMapping(value="/kerberos-startup-state") @ResponseStatus(HttpStatus.OK) public Result getKerberosStartupState(@ApiIgnore @RequestAttribute(value = Constants.SESSION_USER) User loginUser){ - logger.info("login user {},get kerberos startup state : {}", loginUser.getUserName()); + logger.info("login user {}", loginUser.getUserName()); try{ // if upload resource is HDFS and kerberos startup is true , else false return success(Status.SUCCESS.getMsg(), CommonUtils.getKerberosStartupState()); diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/TenantController.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/TenantController.java index 6b0a4ec496..afdb80bd2c 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/TenantController.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/TenantController.java @@ -137,7 +137,7 @@ public class TenantController extends BaseController{ @GetMapping(value="/list") @ResponseStatus(HttpStatus.OK) public Result queryTenantlist(@ApiIgnore @RequestAttribute(value = Constants.SESSION_USER) User loginUser){ - logger.info("login user {}, query tenant list"); + logger.info("login user {}, query tenant list", loginUser.getUserName()); try{ Map result = tenantService.queryTenantList(loginUser); return returnDataList(result); diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/UsersController.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/UsersController.java index b89dd0fa2c..42f89237ab 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/UsersController.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/controller/UsersController.java @@ -324,7 +324,7 @@ public class UsersController extends BaseController{ @GetMapping(value="/get-user-info") @ResponseStatus(HttpStatus.OK) public Result getUserInfo(@ApiIgnore @RequestAttribute(value = Constants.SESSION_USER) User loginUser){ - logger.info("login user {},get user info : {}", loginUser.getUserName()); + logger.info("login user {},get user info", loginUser.getUserName()); try{ Map result = usersService.getUserInfo(loginUser); return returnDataList(result); @@ -344,7 +344,7 @@ public class UsersController extends BaseController{ @GetMapping(value="/list") @ResponseStatus(HttpStatus.OK) public Result listUser(@ApiIgnore @RequestAttribute(value = Constants.SESSION_USER) User loginUser){ - logger.info("login user {}, user list"); + logger.info("login user {}, user list", loginUser.getUserName()); try{ Map result = usersService.queryAllGeneralUsers(loginUser); return returnDataList(result); @@ -364,7 +364,7 @@ public class UsersController extends BaseController{ @GetMapping(value="/list-all") @ResponseStatus(HttpStatus.OK) public Result listAll(@RequestAttribute(value = Constants.SESSION_USER) User loginUser){ - logger.info("login user {}, user list"); + logger.info("login user {}, user list", loginUser.getUserName()); try{ Map result = usersService.queryUserList(loginUser); return returnDataList(result); diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/SchedulerService.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/SchedulerService.java index 042e16f9f9..ee97747ec2 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/SchedulerService.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/SchedulerService.java @@ -374,12 +374,12 @@ public class SchedulerService extends BaseService { try { switch (scheduleStatus) { case ONLINE: { - logger.info("Call master client set schedule online, project id: {}, flow id: {},host: {}, port: {}", project.getId(), processDefinition.getId(), masterServers); + logger.info("Call master client set schedule online, project id: {}, flow id: {},host: {}", project.getId(), processDefinition.getId(), masterServers); setSchedule(project.getId(), id); break; } case OFFLINE: { - logger.info("Call master client set schedule offline, project id: {}, flow id: {},host: {}, port: {}", project.getId(), processDefinition.getId(), masterServers); + logger.info("Call master client set schedule offline, project id: {}, flow id: {},host: {}", project.getId(), processDefinition.getId(), masterServers); deleteSchedule(project.getId(), id); break; } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/HadoopUtils.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/HadoopUtils.java index 072266fb7f..ef933edfbb 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/HadoopUtils.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/HadoopUtils.java @@ -116,7 +116,7 @@ public class HadoopUtils implements Closeable { if(StringUtils.isNotBlank(defaultFSProp)){ Map fsRelatedProps = PropertyUtils.getPrefixedProperties("fs."); configuration.set(Constants.FS_DEFAULTFS,defaultFSProp); - fsRelatedProps.entrySet().stream().forEach(entry -> configuration.set(entry.getKey(), entry.getValue())); + fsRelatedProps.forEach((key, value) -> configuration.set(key, value)); }else{ logger.error("property:{} can not to be empty, please set!"); throw new RuntimeException("property:{} can not to be empty, please set!"); diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/HttpUtils.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/HttpUtils.java index 7bb93d8c46..c163dcab3d 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/HttpUtils.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/HttpUtils.java @@ -67,7 +67,7 @@ public class HttpUtils { logger.warn("http entity is null"); } }else{ - logger.error("htt get:{} response status code is not 200!"); + logger.error("http get:{} response status code is not 200!", response.getStatusLine().getStatusCode()); } }catch (Exception e){ logger.error(e.getMessage(),e); From 216154dc068a6a5798860cfe32069f8ddf2097ab Mon Sep 17 00:00:00 2001 From: Yelli <51317527+Yeleights@users.noreply.github.com> Date: Fri, 3 Jan 2020 11:46:51 +0800 Subject: [PATCH 10/40] fix sonar bug: null check & clean throw exception (#1690) * fix sonar bug: null check & clean throw exception * fix sonar bug: null check & clean throw exception --- .../api/service/ProcessInstanceService.java | 11 +++++------ .../api/service/TaskInstanceService.java | 7 +++---- .../common/job/db/ClickHouseDataSource.java | 1 - .../common/job/db/DB2ServerDataSource.java | 1 - .../common/job/db/HiveDataSource.java | 1 - .../common/job/db/MySQLDataSource.java | 1 - .../common/job/db/OracleDataSource.java | 1 - .../common/job/db/PostgreDataSource.java | 1 - .../common/job/db/SQLServerDataSource.java | 1 - .../common/job/db/SparkDataSource.java | 1 - .../dolphinscheduler/common/model/DateInterval.java | 1 + .../common/model/TaskNodeRelation.java | 1 + .../common/utils/ConnectionUtils.java | 3 --- .../dolphinscheduler/dao/upgrade/UpgradeDao.java | 12 +++++++++--- 14 files changed, 19 insertions(+), 24 deletions(-) diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/ProcessInstanceService.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/ProcessInstanceService.java index db9a6ef8ff..5eae886680 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/ProcessInstanceService.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/ProcessInstanceService.java @@ -195,12 +195,11 @@ public class ProcessInstanceService extends BaseDAGService { processInstance.setDuration(DateUtils.differSec(processInstance.getStartTime(),processInstance.getEndTime())); } - Set exclusionSet = new HashSet(){{ - add(Constants.CLASS); - add("locations"); - add("connects"); - add("processInstanceJson"); - }}; + Set exclusionSet = new HashSet(); + exclusionSet.add(Constants.CLASS); + exclusionSet.add("locations"); + exclusionSet.add("connects"); + exclusionSet.add("processInstanceJson"); PageInfo pageInfo = new PageInfo(pageNo, pageSize); pageInfo.setTotalCount((int) processInstanceList.getTotal()); diff --git a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/TaskInstanceService.java b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/TaskInstanceService.java index c1c5acd44c..17b698be70 100644 --- a/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/TaskInstanceService.java +++ b/dolphinscheduler-api/src/main/java/org/apache/dolphinscheduler/api/service/TaskInstanceService.java @@ -116,10 +116,9 @@ public class TaskInstanceService extends BaseService { page, project.getId(), processInstanceId, searchVal, taskName, statusArray, host, start, end ); PageInfo pageInfo = new PageInfo(pageNo, pageSize); - Set exclusionSet = new HashSet(){{ - add(Constants.CLASS); - add("taskJson"); - }}; + Set exclusionSet = new HashSet<>(); + exclusionSet.add(Constants.CLASS); + exclusionSet.add("taskJson"); List taskInstanceList = taskInstanceIPage.getRecords(); for(TaskInstance taskInstance : taskInstanceList){ taskInstance.setDuration(DateUtils.differSec(taskInstance.getStartTime(), diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/ClickHouseDataSource.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/ClickHouseDataSource.java index dfcda99141..457ddb0a1d 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/ClickHouseDataSource.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/ClickHouseDataSource.java @@ -66,7 +66,6 @@ public class ClickHouseDataSource extends BaseDataSource { con.close(); } catch (SQLException e) { logger.error("ClickHouse datasource try conn close conn error", e); - throw e; } } } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/DB2ServerDataSource.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/DB2ServerDataSource.java index e6d7b94b51..44ee200c5d 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/DB2ServerDataSource.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/DB2ServerDataSource.java @@ -66,7 +66,6 @@ public class DB2ServerDataSource extends BaseDataSource { con.close(); } catch (SQLException e) { logger.error("DB2 Server datasource try conn close conn error", e); - throw e; } } } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/HiveDataSource.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/HiveDataSource.java index 2970354e6b..101efae793 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/HiveDataSource.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/HiveDataSource.java @@ -69,7 +69,6 @@ public class HiveDataSource extends BaseDataSource { con.close(); } catch (SQLException e) { logger.error("hive datasource try conn close conn error", e); - throw e; } } } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/MySQLDataSource.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/MySQLDataSource.java index f50de0dee3..3cf2b2ce8c 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/MySQLDataSource.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/MySQLDataSource.java @@ -64,7 +64,6 @@ public class MySQLDataSource extends BaseDataSource { con.close(); } catch (SQLException e) { logger.error("Mysql datasource try conn close conn error", e); - throw e; } } } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/OracleDataSource.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/OracleDataSource.java index ddc30d939a..551c7823cb 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/OracleDataSource.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/OracleDataSource.java @@ -66,7 +66,6 @@ public class OracleDataSource extends BaseDataSource { con.close(); } catch (SQLException e) { logger.error("Oracle datasource try conn close conn error", e); - throw e; } } } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/PostgreDataSource.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/PostgreDataSource.java index ecc29c73e8..5241b4c7ef 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/PostgreDataSource.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/PostgreDataSource.java @@ -68,7 +68,6 @@ public class PostgreDataSource extends BaseDataSource { con.close(); } catch (SQLException e) { logger.error("Postgre datasource try conn close conn error", e); - throw e; } } } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/SQLServerDataSource.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/SQLServerDataSource.java index 7a51ff7a45..fe398eb88b 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/SQLServerDataSource.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/SQLServerDataSource.java @@ -62,7 +62,6 @@ public class SQLServerDataSource extends BaseDataSource { con.close(); } catch (SQLException e) { logger.error("SQL Server datasource try conn close conn error", e); - throw e; } } } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/SparkDataSource.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/SparkDataSource.java index 638664c47d..eb455124de 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/SparkDataSource.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/job/db/SparkDataSource.java @@ -71,7 +71,6 @@ public class SparkDataSource extends BaseDataSource { con.close(); } catch (SQLException e) { logger.error("Spark datasource try conn close conn error", e); - throw e; } } } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/model/DateInterval.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/model/DateInterval.java index e3bca6ecbf..3519d5c535 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/model/DateInterval.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/model/DateInterval.java @@ -59,4 +59,5 @@ public class DateInterval { public void setEndTime(Date endTime) { this.endTime = endTime; } + } diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/model/TaskNodeRelation.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/model/TaskNodeRelation.java index 91436399c5..2fbc0e1654 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/model/TaskNodeRelation.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/model/TaskNodeRelation.java @@ -53,6 +53,7 @@ public class TaskNodeRelation { } + @Override public boolean equals(Object o){ if (!(o instanceof TaskNodeRelation)) { return false; diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/ConnectionUtils.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/ConnectionUtils.java index 2bbb45b64c..c1c3ff5d57 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/ConnectionUtils.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/ConnectionUtils.java @@ -51,7 +51,6 @@ public class ConnectionUtils { } } catch (SQLException e) { logger.error(e.getMessage(),e); - throw new RuntimeException(e); } finally { try { if (stmt != null) { @@ -60,7 +59,6 @@ public class ConnectionUtils { } } catch (SQLException e) { logger.error(e.getMessage(),e); - throw new RuntimeException(e); } finally { try { if (conn != null) { @@ -69,7 +67,6 @@ public class ConnectionUtils { } } catch (SQLException e) { logger.error(e.getMessage(),e); - throw new RuntimeException(e); } } } diff --git a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/upgrade/UpgradeDao.java b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/upgrade/UpgradeDao.java index 96338248b8..aed93038f5 100644 --- a/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/upgrade/UpgradeDao.java +++ b/dolphinscheduler-dao/src/main/java/org/apache/dolphinscheduler/dao/upgrade/UpgradeDao.java @@ -155,7 +155,9 @@ public abstract class UpgradeDao extends AbstractBaseDao { throw new RuntimeException(e.getMessage(),e); } catch (Exception e) { try { - conn.rollback(); + if (null != conn) { + conn.rollback(); + } } catch (SQLException e1) { logger.error(e1.getMessage(),e1); } @@ -312,7 +314,9 @@ public abstract class UpgradeDao extends AbstractBaseDao { throw new RuntimeException(e.getMessage(),e); } catch (SQLException e) { try { - conn.rollback(); + if (null != conn) { + conn.rollback(); + } } catch (SQLException e1) { logger.error(e1.getMessage(),e1); } @@ -320,7 +324,9 @@ public abstract class UpgradeDao extends AbstractBaseDao { throw new RuntimeException(e.getMessage(),e); } catch (Exception e) { try { - conn.rollback(); + if (null != conn) { + conn.rollback(); + } } catch (SQLException e1) { logger.error(e1.getMessage(),e1); } From 0e66c71f38958c7d7013b6137cf3dcdd6ebe1d15 Mon Sep 17 00:00:00 2001 From: Tboy Date: Fri, 3 Jan 2020 16:10:18 +0800 Subject: [PATCH 11/40] remove commons-collections4/commons-logging/commons-io dependancy for API module (#1693) * remove commons-lang3 dependancy * remove commons-lang3 and commons-lang dependancy * remove commons-collections4/commons-logging/commons-io dependancy * tmp add commons-lang3 for compile --- dolphinscheduler-alert/pom.xml | 21 --------- .../alert/runner/AlertSender.java | 4 +- .../alert/utils/EnterpriseWeChatUtils.java | 2 +- .../alert/utils/FuncUtils.java | 2 +- .../alert/utils/JSONUtils.java | 2 +- .../alert/utils/MailUtils.java | 11 +++-- .../alert/utils/PropertyUtils.java | 4 +- .../common/utils/IOUtils.java | 45 +++++++++++++++++++ .../common/utils/StringUtils.java | 30 +++++++++++++ dolphinscheduler-server/pom.xml | 5 +++ 10 files changed, 92 insertions(+), 34 deletions(-) create mode 100644 dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/IOUtils.java diff --git a/dolphinscheduler-alert/pom.xml b/dolphinscheduler-alert/pom.xml index ead46f5e8d..c02368418a 100644 --- a/dolphinscheduler-alert/pom.xml +++ b/dolphinscheduler-alert/pom.xml @@ -67,21 +67,6 @@ slf4j-api - - org.apache.commons - commons-collections4 - - - - commons-logging - commons-logging - - - - org.apache.commons - commons-lang3 - - com.google.guava guava @@ -92,12 +77,6 @@ logback-classic - - commons-io - commons-io - - - org.apache.poi diff --git a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/runner/AlertSender.java b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/runner/AlertSender.java index 5e0c2545f8..3703b5d64a 100644 --- a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/runner/AlertSender.java +++ b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/runner/AlertSender.java @@ -16,14 +16,14 @@ */ package org.apache.dolphinscheduler.alert.runner; -import org.apache.commons.collections4.CollectionUtils; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.alert.manager.EmailManager; import org.apache.dolphinscheduler.alert.manager.EnterpriseWeChatManager; import org.apache.dolphinscheduler.alert.utils.Constants; import org.apache.dolphinscheduler.alert.utils.EnterpriseWeChatUtils; import org.apache.dolphinscheduler.common.enums.AlertStatus; import org.apache.dolphinscheduler.common.enums.AlertType; +import org.apache.dolphinscheduler.common.utils.CollectionUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.AlertDao; import org.apache.dolphinscheduler.dao.entity.Alert; import org.apache.dolphinscheduler.dao.entity.User; diff --git a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/EnterpriseWeChatUtils.java b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/EnterpriseWeChatUtils.java index 15e7e18855..a4c3720581 100644 --- a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/EnterpriseWeChatUtils.java +++ b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/EnterpriseWeChatUtils.java @@ -17,11 +17,11 @@ package org.apache.dolphinscheduler.alert.utils; import org.apache.dolphinscheduler.common.enums.ShowType; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.entity.Alert; import com.alibaba.fastjson.JSON; import com.google.common.reflect.TypeToken; -import org.apache.commons.lang3.StringUtils; import org.apache.http.HttpEntity; import org.apache.http.client.methods.CloseableHttpResponse; import org.apache.http.client.methods.HttpGet; diff --git a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/FuncUtils.java b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/FuncUtils.java index 186a983e05..dd6ca4b8a6 100644 --- a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/FuncUtils.java +++ b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/FuncUtils.java @@ -16,7 +16,7 @@ */ package org.apache.dolphinscheduler.alert.utils; -import org.apache.commons.lang.StringUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; public class FuncUtils { diff --git a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/JSONUtils.java b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/JSONUtils.java index a88574f0e7..1cd9f490b2 100644 --- a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/JSONUtils.java +++ b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/JSONUtils.java @@ -18,7 +18,7 @@ package org.apache.dolphinscheduler.alert.utils; import com.alibaba.fastjson.JSONArray; import com.alibaba.fastjson.JSONObject; -import org.apache.commons.lang3.StringUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; diff --git a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/MailUtils.java b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/MailUtils.java index d6edde240b..b7634fe567 100644 --- a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/MailUtils.java +++ b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/MailUtils.java @@ -21,11 +21,11 @@ import freemarker.cache.StringTemplateLoader; import freemarker.template.Configuration; import freemarker.template.Template; import freemarker.template.TemplateException; -import org.apache.commons.collections4.CollectionUtils; -import org.apache.commons.io.IOUtils; -import org.apache.commons.lang3.StringUtils; import org.apache.commons.mail.EmailException; import org.apache.commons.mail.HtmlEmail; +import org.apache.dolphinscheduler.common.utils.CollectionUtils; +import org.apache.dolphinscheduler.common.utils.IOUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; import org.springframework.util.ResourceUtils; @@ -35,7 +35,6 @@ import javax.mail.internet.*; import java.io.*; import java.util.*; -import static org.apache.dolphinscheduler.alert.utils.PropertyUtils.getInt; /** @@ -422,8 +421,8 @@ public class MailUtils { * @param e the exception */ private static void handleException(Collection receivers, Map retMap, Exception e) { - logger.error("Send email to {} failed", StringUtils.join(",", receivers), e); - retMap.put(Constants.MESSAGE, "Send email to {" + StringUtils.join(",", receivers) + "} failed," + e.toString()); + logger.error("Send email to {} failed {}", receivers, e); + retMap.put(Constants.MESSAGE, "Send email to {" + StringUtils.join(receivers, ",") + "} failed," + e.toString()); } /** diff --git a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/PropertyUtils.java b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/PropertyUtils.java index 14ec4149eb..4367fbb4a8 100644 --- a/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/PropertyUtils.java +++ b/dolphinscheduler-alert/src/main/java/org/apache/dolphinscheduler/alert/utils/PropertyUtils.java @@ -16,8 +16,8 @@ */ package org.apache.dolphinscheduler.alert.utils; -import org.apache.commons.io.IOUtils; -import org.apache.commons.lang.StringUtils; +import org.apache.dolphinscheduler.common.utils.IOUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/IOUtils.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/IOUtils.java new file mode 100644 index 0000000000..73df158aa3 --- /dev/null +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/IOUtils.java @@ -0,0 +1,45 @@ + +/* + * Licensed to the Apache Software Foundation (ASF) under one or more + * contributor license agreements. See the NOTICE file distributed with + * this work for additional information regarding copyright ownership. + * The ASF licenses this file to You under the Apache License, Version 2.0 + * (the "License"); you may not use this file except in compliance with + * the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.dolphinscheduler.common.utils; + + +import java.io.IOException; +import java.io.InputStream; +import java.io.InputStreamReader; + +public class IOUtils { + + public static void closeQuietly(InputStream fis){ + if(fis != null){ + try { + fis.close(); + } catch (IOException ignore) { + } + } + } + + public static void closeQuietly(InputStreamReader reader){ + if(reader != null){ + try { + reader.close(); + } catch (IOException ignore) { + } + } + } +} diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/StringUtils.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/StringUtils.java index 64dd4f3dcd..12b75fb0e5 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/StringUtils.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/StringUtils.java @@ -18,6 +18,8 @@ package org.apache.dolphinscheduler.common.utils; import java.nio.charset.StandardCharsets; +import java.util.Iterator; +import java.util.Objects; import java.util.regex.Pattern; @@ -122,4 +124,32 @@ public class StringUtils { } return false; } + + public static String join(final Iterable iterable, final String separator){ + Iterator iterator = iterable.iterator(); + if (iterator == null) { + return null; + } + if (!iterator.hasNext()) { + return EMPTY; + } + final Object first = iterator.next(); + if (!iterable.iterator().hasNext()) { + return Objects.toString(first, ""); + } + final StringBuilder buf = new StringBuilder(64); + if (first != null) { + buf.append(first); + } + while (iterator.hasNext()) { + if (separator != null) { + buf.append(separator); + } + final Object obj = iterator.next(); + if (obj != null) { + buf.append(obj); + } + } + return buf.toString(); + } } diff --git a/dolphinscheduler-server/pom.xml b/dolphinscheduler-server/pom.xml index 2ccc880a41..882ff09b59 100644 --- a/dolphinscheduler-server/pom.xml +++ b/dolphinscheduler-server/pom.xml @@ -111,6 +111,11 @@ dolphinscheduler-alert + + org.apache.commons + commons-lang3 + + From 3477d28de872c7a7a7d05146665a1dc9f095cb64 Mon Sep 17 00:00:00 2001 From: break60 <790061044@qq.com> Date: Fri, 3 Jan 2020 17:50:05 +0800 Subject: [PATCH 12/40] Fixed DAG zoom in and zoom out nodes separated from arrows #1679 (#1680) * Fix api url * Fixed DAG zoom in and zoom out nodes separated from arrows --- .../src/js/conf/home/pages/dag/_source/dag.scss | 2 +- .../src/js/conf/home/pages/dag/_source/dag.vue | 9 ++++++++- 2 files changed, 9 insertions(+), 2 deletions(-) diff --git a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.scss b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.scss index 7fdb5e64f2..88a8bdad39 100644 --- a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.scss +++ b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.scss @@ -309,13 +309,13 @@ svg path:hover { //min-width: calc(100% - 220px); width: 8000px; height: 5000px; - position: relative; svg:not(:root){ z-index: 11; } } .jtk-demo-canvas { + position: relative; height: 100%; } diff --git a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.vue b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.vue index 8b1c353d41..3b70f01df5 100644 --- a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.vue +++ b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.vue @@ -161,7 +161,7 @@ // DAG automatic layout dagAutomaticLayout() { $('#canvas').html('') - + // Destroy round robin Dag.init({ dag: this, @@ -187,6 +187,13 @@ }) if (this.tasks.length) { Dag.backfill(true) + if (this.type === 'instance') { + this._getTaskState(false).then(res => {}) + // Round robin acquisition status + this.setIntervalP = setInterval(() => { + this._getTaskState(true).then(res => {}) + }, 90000) + } } else { Dag.create() } From 8090f1450471f02868569f3f6c43f0e22788eb1d Mon Sep 17 00:00:00 2001 From: lgcareer <18610854716@163.com> Date: Fri, 3 Jan 2020 18:05:00 +0800 Subject: [PATCH 13/40] get root path from zookeeper config --- .../apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java index 45c6122341..03a6064535 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java @@ -344,7 +344,7 @@ public class TaskQueueZkImpl implements ITaskQueue { * @return */ public String getTasksPath(String key){ - return "/dolphinscheduler" + Constants.SINGLE_SLASH + key; + return zookeeperOperator.getZookeeperConfig().getDsRoot() + Constants.SINGLE_SLASH + key; } } From c21d25edf2b89cbf1c19db0d2da311503652a185 Mon Sep 17 00:00:00 2001 From: elonlo Date: Fri, 3 Jan 2020 18:37:39 +0800 Subject: [PATCH 14/40] Fixed tasks_queue and tasks_kill did not exist in zookeeper #1696 --- .../common/queue/TaskQueueZkImpl.java | 21 ++++++++++++++++++- 1 file changed, 20 insertions(+), 1 deletion(-) diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java index 45c6122341..d022c5e9be 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java @@ -37,8 +37,27 @@ public class TaskQueueZkImpl implements ITaskQueue { private static final Logger logger = LoggerFactory.getLogger(TaskQueueZkImpl.class); + private final ZookeeperOperator zookeeperOperator; + @Autowired - private ZookeeperOperator zookeeperOperator; + public TaskQueueZkImpl(ZookeeperOperator zookeeperOperator) { + this.zookeeperOperator = zookeeperOperator; + + try { + String tasksQueuePath = getTasksPath(Constants.DOLPHINSCHEDULER_TASKS_QUEUE); + String tasksCancelPath = getTasksPath(Constants.DOLPHINSCHEDULER_TASKS_KILL); + + for(String key : new String[]{tasksQueuePath,tasksCancelPath}){ + if(!zookeeperOperator.isExisted(key)){ + zookeeperOperator.persist(key, ""); + logger.info("create tasks queue parent node success : {}", key); + } + } + } catch (Exception e) { + logger.error("create tasks queue parent node failure", e); + } + } + /** * get all tasks from tasks queue From 884237487cb2b58d80eef8b848b6cb4f4209facd Mon Sep 17 00:00:00 2001 From: Tboy Date: Fri, 3 Jan 2020 19:23:05 +0800 Subject: [PATCH 15/40] remove commons-lang3 dependancy, using common-module instead for server module (#1698) * remove commons-lang3 dependancy * remove commons-lang3 and commons-lang dependancy * remove commons-collections4/commons-logging/commons-io dependancy * tmp add commons-lang3 for compile * remove commons-lang3 dependancy, using common-module instead * remove common StringUtils instead of lang3 * add tmp lang3 for compile --- dolphinscheduler-api/pom.xml | 5 +++++ .../dolphinscheduler/common/utils/EnumUtils.java | 12 ++++++++++++ dolphinscheduler-server/pom.xml | 5 ----- .../dolphinscheduler/server/master/MasterServer.java | 2 +- .../server/master/runner/MasterExecThread.java | 1 - .../server/monitor/AbstractMonitor.java | 2 +- .../dolphinscheduler/server/utils/ProcessUtils.java | 2 +- .../dolphinscheduler/server/utils/UDFUtils.java | 2 +- .../server/worker/log/SensitiveDataConverter.java | 2 +- .../server/worker/runner/FetchTaskThread.java | 6 +----- .../server/worker/task/AbstractCommandExecutor.java | 2 +- .../server/worker/task/PythonCommandExecutor.java | 2 +- .../server/worker/task/TaskManager.java | 2 +- .../server/worker/task/flink/FlinkTask.java | 2 +- .../server/worker/task/http/HttpTask.java | 2 +- .../server/worker/task/mr/MapReduceTask.java | 2 +- .../server/worker/task/spark/SparkTask.java | 2 +- .../server/worker/task/sql/SqlTask.java | 7 +------ .../server/worker/task/spark/SparkTaskTest.java | 2 +- 19 files changed, 32 insertions(+), 30 deletions(-) diff --git a/dolphinscheduler-api/pom.xml b/dolphinscheduler-api/pom.xml index 014799b0b6..9d94065e89 100644 --- a/dolphinscheduler-api/pom.xml +++ b/dolphinscheduler-api/pom.xml @@ -162,6 +162,11 @@ commons-collections + + org.apache.commons + commons-lang3 + + org.quartz-scheduler quartz diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/EnumUtils.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/EnumUtils.java index dc2e3a0750..924e8ff719 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/EnumUtils.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/utils/EnumUtils.java @@ -30,4 +30,16 @@ public class EnumUtils { return null; } } + + public static > boolean isValidEnum(final Class enumClass, final String enumName) { + if (enumName == null) { + return false; + } + try { + Enum.valueOf(enumClass, enumName); + return true; + } catch (final IllegalArgumentException ex) { + return false; + } + } } diff --git a/dolphinscheduler-server/pom.xml b/dolphinscheduler-server/pom.xml index 882ff09b59..2ccc880a41 100644 --- a/dolphinscheduler-server/pom.xml +++ b/dolphinscheduler-server/pom.xml @@ -111,11 +111,6 @@ dolphinscheduler-alert - - org.apache.commons - commons-lang3 - - diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/master/MasterServer.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/master/MasterServer.java index 1f5b16a25a..c4cec68683 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/master/MasterServer.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/master/MasterServer.java @@ -16,7 +16,6 @@ */ package org.apache.dolphinscheduler.server.master; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.IStoppable; import org.apache.dolphinscheduler.common.thread.Stopper; @@ -24,6 +23,7 @@ import org.apache.dolphinscheduler.common.thread.ThreadPoolExecutors; import org.apache.dolphinscheduler.common.thread.ThreadUtils; import org.apache.dolphinscheduler.common.utils.OSUtils; import org.apache.dolphinscheduler.common.utils.SpringApplicationContext; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.ProcessDao; import org.apache.dolphinscheduler.server.master.config.MasterConfig; import org.apache.dolphinscheduler.server.master.runner.MasterSchedulerThread; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/master/runner/MasterExecThread.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/master/runner/MasterExecThread.java index 3481b79caa..ade4738884 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/master/runner/MasterExecThread.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/master/runner/MasterExecThread.java @@ -18,7 +18,6 @@ package org.apache.dolphinscheduler.server.master.runner; import com.alibaba.fastjson.JSONObject; import org.apache.commons.io.FileUtils; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.*; import org.apache.dolphinscheduler.common.graph.DAG; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/monitor/AbstractMonitor.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/monitor/AbstractMonitor.java index ab30ce890f..15fef1e95d 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/monitor/AbstractMonitor.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/monitor/AbstractMonitor.java @@ -16,8 +16,8 @@ */ package org.apache.dolphinscheduler.server.monitor; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.utils.CollectionUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; import org.springframework.beans.factory.annotation.Autowired; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/utils/ProcessUtils.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/utils/ProcessUtils.java index 748c8f007e..7cba139cec 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/utils/ProcessUtils.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/utils/ProcessUtils.java @@ -19,10 +19,10 @@ package org.apache.dolphinscheduler.server.utils; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.utils.CommonUtils; import org.apache.dolphinscheduler.common.utils.OSUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.entity.TaskInstance; import org.apache.dolphinscheduler.server.rpc.LogClient; import org.apache.commons.io.FileUtils; -import org.apache.commons.lang.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/utils/UDFUtils.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/utils/UDFUtils.java index 937e35454d..5e2e535cdb 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/utils/UDFUtils.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/utils/UDFUtils.java @@ -18,8 +18,8 @@ package org.apache.dolphinscheduler.server.utils; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.utils.HadoopUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.entity.UdfFunc; -import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import java.text.MessageFormat; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/log/SensitiveDataConverter.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/log/SensitiveDataConverter.java index be8d3d12a0..2406a4da88 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/log/SensitiveDataConverter.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/log/SensitiveDataConverter.java @@ -20,8 +20,8 @@ package org.apache.dolphinscheduler.server.worker.log; import ch.qos.logback.classic.pattern.MessageConverter; import ch.qos.logback.classic.spi.ILoggingEvent; import lombok.extern.slf4j.Slf4j; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.Constants; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.server.utils.SensitiveLogUtil; import java.util.regex.Matcher; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/runner/FetchTaskThread.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/runner/FetchTaskThread.java index ae67716da2..60cfd82cdd 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/runner/FetchTaskThread.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/runner/FetchTaskThread.java @@ -16,16 +16,12 @@ */ package org.apache.dolphinscheduler.server.worker.runner; -import org.apache.commons.lang3.StringUtils; import org.apache.curator.framework.recipes.locks.InterProcessMutex; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.queue.ITaskQueue; import org.apache.dolphinscheduler.common.thread.Stopper; import org.apache.dolphinscheduler.common.thread.ThreadUtils; -import org.apache.dolphinscheduler.common.utils.CollectionUtils; -import org.apache.dolphinscheduler.common.utils.FileUtils; -import org.apache.dolphinscheduler.common.utils.OSUtils; -import org.apache.dolphinscheduler.common.utils.SpringApplicationContext; +import org.apache.dolphinscheduler.common.utils.*; import org.apache.dolphinscheduler.common.zk.AbstractZKClient; import org.apache.dolphinscheduler.dao.ProcessDao; import org.apache.dolphinscheduler.dao.entity.TaskInstance; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/AbstractCommandExecutor.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/AbstractCommandExecutor.java index 32a92d1b8f..715fda108e 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/AbstractCommandExecutor.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/AbstractCommandExecutor.java @@ -20,11 +20,11 @@ import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.ExecutionStatus; import org.apache.dolphinscheduler.common.thread.ThreadUtils; import org.apache.dolphinscheduler.common.utils.HadoopUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.ProcessDao; import org.apache.dolphinscheduler.dao.entity.TaskInstance; import org.apache.dolphinscheduler.server.utils.LoggerUtils; import org.apache.dolphinscheduler.server.utils.ProcessUtils; -import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import java.io.*; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/PythonCommandExecutor.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/PythonCommandExecutor.java index 17413999e2..c1ff89d4cf 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/PythonCommandExecutor.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/PythonCommandExecutor.java @@ -18,7 +18,7 @@ package org.apache.dolphinscheduler.server.worker.task; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.utils.FileUtils; -import org.apache.commons.lang3.StringUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.slf4j.Logger; import org.slf4j.LoggerFactory; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/TaskManager.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/TaskManager.java index ce27e1974c..310d9cad9a 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/TaskManager.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/TaskManager.java @@ -18,6 +18,7 @@ package org.apache.dolphinscheduler.server.worker.task; import org.apache.dolphinscheduler.common.enums.TaskType; +import org.apache.dolphinscheduler.common.utils.EnumUtils; import org.apache.dolphinscheduler.server.worker.task.dependent.DependentTask; import org.apache.dolphinscheduler.server.worker.task.flink.FlinkTask; import org.apache.dolphinscheduler.server.worker.task.http.HttpTask; @@ -27,7 +28,6 @@ import org.apache.dolphinscheduler.server.worker.task.python.PythonTask; import org.apache.dolphinscheduler.server.worker.task.shell.ShellTask; import org.apache.dolphinscheduler.server.worker.task.spark.SparkTask; import org.apache.dolphinscheduler.server.worker.task.sql.SqlTask; -import org.apache.commons.lang3.EnumUtils; import org.slf4j.Logger; /** diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/flink/FlinkTask.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/flink/FlinkTask.java index 58ca74a204..0fa9e11ce5 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/flink/FlinkTask.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/flink/FlinkTask.java @@ -21,12 +21,12 @@ import org.apache.dolphinscheduler.common.task.AbstractParameters; import org.apache.dolphinscheduler.common.task.flink.FlinkParameters; import org.apache.dolphinscheduler.common.utils.JSONUtils; import org.apache.dolphinscheduler.common.utils.ParameterUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.entity.ProcessInstance; import org.apache.dolphinscheduler.server.utils.FlinkArgsUtils; import org.apache.dolphinscheduler.server.utils.ParamUtils; import org.apache.dolphinscheduler.server.worker.task.AbstractYarnTask; import org.apache.dolphinscheduler.server.worker.task.TaskProps; -import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import java.util.ArrayList; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/http/HttpTask.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/http/HttpTask.java index 39bcd922c8..97e6cb7bee 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/http/HttpTask.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/http/HttpTask.java @@ -19,7 +19,6 @@ package org.apache.dolphinscheduler.server.worker.task.http; import com.alibaba.fastjson.JSONObject; import org.apache.commons.io.Charsets; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.HttpMethod; import org.apache.dolphinscheduler.common.enums.HttpParametersType; @@ -30,6 +29,7 @@ import org.apache.dolphinscheduler.common.task.http.HttpParameters; import org.apache.dolphinscheduler.common.utils.DateUtils; import org.apache.dolphinscheduler.common.utils.ParameterUtils; import org.apache.dolphinscheduler.common.utils.SpringApplicationContext; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.dao.ProcessDao; import org.apache.dolphinscheduler.dao.entity.ProcessInstance; import org.apache.dolphinscheduler.server.utils.ParamUtils; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/mr/MapReduceTask.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/mr/MapReduceTask.java index 4da3d2c656..b86ff9952e 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/mr/MapReduceTask.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/mr/MapReduceTask.java @@ -23,10 +23,10 @@ import org.apache.dolphinscheduler.common.task.AbstractParameters; import org.apache.dolphinscheduler.common.task.mr.MapreduceParameters; import org.apache.dolphinscheduler.common.utils.JSONUtils; import org.apache.dolphinscheduler.common.utils.ParameterUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.server.utils.ParamUtils; import org.apache.dolphinscheduler.server.worker.task.AbstractYarnTask; import org.apache.dolphinscheduler.server.worker.task.TaskProps; -import org.apache.commons.lang3.StringUtils; import org.slf4j.Logger; import java.util.ArrayList; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/spark/SparkTask.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/spark/SparkTask.java index 34f7d13ca8..203c0fe146 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/spark/SparkTask.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/spark/SparkTask.java @@ -16,13 +16,13 @@ */ package org.apache.dolphinscheduler.server.worker.task.spark; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.enums.SparkVersion; import org.apache.dolphinscheduler.common.process.Property; import org.apache.dolphinscheduler.common.task.AbstractParameters; import org.apache.dolphinscheduler.common.task.spark.SparkParameters; import org.apache.dolphinscheduler.common.utils.JSONUtils; import org.apache.dolphinscheduler.common.utils.ParameterUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.server.utils.ParamUtils; import org.apache.dolphinscheduler.server.utils.SparkArgsUtils; import org.apache.dolphinscheduler.server.worker.task.AbstractYarnTask; diff --git a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/sql/SqlTask.java b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/sql/SqlTask.java index 08a90c62ce..d88d39cb91 100644 --- a/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/sql/SqlTask.java +++ b/dolphinscheduler-server/src/main/java/org/apache/dolphinscheduler/server/worker/task/sql/SqlTask.java @@ -19,8 +19,6 @@ package org.apache.dolphinscheduler.server.worker.task.sql; import com.alibaba.fastjson.JSONArray; import com.alibaba.fastjson.JSONObject; import com.alibaba.fastjson.serializer.SerializerFeature; -import org.apache.commons.lang.StringUtils; -import org.apache.commons.lang3.EnumUtils; import org.apache.dolphinscheduler.alert.utils.MailUtils; import org.apache.dolphinscheduler.common.Constants; import org.apache.dolphinscheduler.common.enums.ShowType; @@ -33,10 +31,7 @@ import org.apache.dolphinscheduler.common.task.AbstractParameters; import org.apache.dolphinscheduler.common.task.sql.SqlBinds; import org.apache.dolphinscheduler.common.task.sql.SqlParameters; import org.apache.dolphinscheduler.common.task.sql.SqlType; -import org.apache.dolphinscheduler.common.utils.CollectionUtils; -import org.apache.dolphinscheduler.common.utils.CommonUtils; -import org.apache.dolphinscheduler.common.utils.ParameterUtils; -import org.apache.dolphinscheduler.common.utils.SpringApplicationContext; +import org.apache.dolphinscheduler.common.utils.*; import org.apache.dolphinscheduler.dao.AlertDao; import org.apache.dolphinscheduler.dao.ProcessDao; import org.apache.dolphinscheduler.dao.entity.DataSource; diff --git a/dolphinscheduler-server/src/test/java/org/apache/dolphinscheduler/server/worker/task/spark/SparkTaskTest.java b/dolphinscheduler-server/src/test/java/org/apache/dolphinscheduler/server/worker/task/spark/SparkTaskTest.java index b502e13bc6..a18e0b2a9d 100644 --- a/dolphinscheduler-server/src/test/java/org/apache/dolphinscheduler/server/worker/task/spark/SparkTaskTest.java +++ b/dolphinscheduler-server/src/test/java/org/apache/dolphinscheduler/server/worker/task/spark/SparkTaskTest.java @@ -16,12 +16,12 @@ */ package org.apache.dolphinscheduler.server.worker.task.spark; -import org.apache.commons.lang3.StringUtils; import org.apache.dolphinscheduler.common.enums.SparkVersion; import org.apache.dolphinscheduler.common.process.Property; import org.apache.dolphinscheduler.common.task.spark.SparkParameters; import org.apache.dolphinscheduler.common.utils.JSONUtils; import org.apache.dolphinscheduler.common.utils.ParameterUtils; +import org.apache.dolphinscheduler.common.utils.StringUtils; import org.apache.dolphinscheduler.server.utils.ParamUtils; import org.apache.dolphinscheduler.server.utils.SparkArgsUtils; import org.apache.dolphinscheduler.server.worker.task.TaskProps; From cae656c1bc230e9097a94ab88b6cef379cbb51e0 Mon Sep 17 00:00:00 2001 From: elonlo Date: Fri, 3 Jan 2020 19:25:10 +0800 Subject: [PATCH 16/40] uniform naming --- .../common/queue/TaskQueueZkImpl.java | 20 +++++++++---------- 1 file changed, 10 insertions(+), 10 deletions(-) diff --git a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java index d022c5e9be..f4e865e4f5 100644 --- a/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java +++ b/dolphinscheduler-common/src/main/java/org/apache/dolphinscheduler/common/queue/TaskQueueZkImpl.java @@ -45,10 +45,10 @@ public class TaskQueueZkImpl implements ITaskQueue { try { String tasksQueuePath = getTasksPath(Constants.DOLPHINSCHEDULER_TASKS_QUEUE); - String tasksCancelPath = getTasksPath(Constants.DOLPHINSCHEDULER_TASKS_KILL); + String tasksKillPath = getTasksPath(Constants.DOLPHINSCHEDULER_TASKS_KILL); - for(String key : new String[]{tasksQueuePath,tasksCancelPath}){ - if(!zookeeperOperator.isExisted(key)){ + for (String key : new String[]{tasksQueuePath,tasksKillPath}){ + if (!zookeeperOperator.isExisted(key)){ zookeeperOperator.persist(key, ""); logger.info("create tasks queue parent node success : {}", key); } @@ -340,20 +340,20 @@ public class TaskQueueZkImpl implements ITaskQueue { public void delete(){ try { String tasksQueuePath = getTasksPath(Constants.DOLPHINSCHEDULER_TASKS_QUEUE); - String tasksCancelPath = getTasksPath(Constants.DOLPHINSCHEDULER_TASKS_KILL); + String tasksKillPath = getTasksPath(Constants.DOLPHINSCHEDULER_TASKS_KILL); - for(String taskQueuePath : new String[]{tasksQueuePath,tasksCancelPath}){ - if(zookeeperOperator.isExisted(taskQueuePath)){ - List list = zookeeperOperator.getChildrenKeys(taskQueuePath); + for (String key : new String[]{tasksQueuePath,tasksKillPath}){ + if (zookeeperOperator.isExisted(key)){ + List list = zookeeperOperator.getChildrenKeys(key); for (String task : list) { - zookeeperOperator.remove(taskQueuePath + Constants.SINGLE_SLASH + task); - logger.info("delete task from tasks queue : {}/{} ",taskQueuePath,task); + zookeeperOperator.remove(key + Constants.SINGLE_SLASH + task); + logger.info("delete task from tasks queue : {}/{} ", key, task); } } } } catch (Exception e) { - logger.error("delete all tasks in tasks queue failure",e); + logger.error("delete all tasks in tasks queue failure", e); } } From cfe174293f1a0076340a9ebd84955a6206692257 Mon Sep 17 00:00:00 2001 From: zhukai Date: Fri, 3 Jan 2020 21:07:22 +0800 Subject: [PATCH 17/40] Improve the user experience in the task definition page (#1678) define the architecture (include actions.js, dag.vue,formModel.vue,mutations.js, and state.js) and implement one of the tasks , such as http.vue. --- .../js/conf/home/pages/dag/_source/dag.vue | 12 ++- .../pages/dag/_source/formModel/formModel.vue | 90 +++++++++++++++---- .../dag/_source/formModel/tasks/http.vue | 19 ++++ .../src/js/conf/home/store/dag/mutations.js | 17 ++++ .../src/js/conf/home/store/dag/state.js | 2 + 5 files changed, 121 insertions(+), 19 deletions(-) diff --git a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.vue b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.vue index 3b70f01df5..01da782246 100644 --- a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.vue +++ b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/dag.vue @@ -156,8 +156,8 @@ }, methods: { ...mapActions('dag', ['saveDAGchart', 'updateInstance', 'updateDefinition', 'getTaskState']), - ...mapMutations('dag', ['addTasks', 'resetParams', 'setIsEditDag', 'setName']), - + ...mapMutations('dag', ['addTasks', 'cacheTasks', 'resetParams', 'setIsEditDag', 'setName']), + // DAG automatic layout dagAutomaticLayout() { $('#canvas').html('') @@ -495,6 +495,14 @@ removeNodesEvent(fromThis) }, 100) }, + /** + * Cache the item + * @param item + * @param fromThis + */ + cacheTaskInfo({item, fromThis}) { + self.cacheTasks(item) + }, close ({ flag, fromThis }) { // Edit status does not allow deletion of nodes if (flag) { diff --git a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/formModel.vue b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/formModel.vue index 59307fa4a1..57c7ad4c18 100644 --- a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/formModel.vue +++ b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/formModel.vue @@ -16,7 +16,7 @@ */ diff --git a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/_source/statementList.vue b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/_source/statementList.vue index 226e23219a..ecd10a9411 100644 --- a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/_source/statementList.vue +++ b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/_source/statementList.vue @@ -32,18 +32,18 @@ - + - + - + diff --git a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/dependent.vue b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/dependent.vue index 785a4fe193..cca9ec7003 100644 --- a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/dependent.vue +++ b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/dependent.vue @@ -23,8 +23,8 @@ - - + +
@@ -40,13 +40,13 @@ @click="!isDetails && _setRelation($index)"> {{el.relation === 'AND' ? $t('and') : $t('or')}} - - + - +
diff --git a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/sql.vue b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/sql.vue index 9cef7bb801..7ab032a798 100644 --- a/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/sql.vue +++ b/dolphinscheduler-ui/src/js/conf/home/pages/dag/_source/formModel/tasks/sql.vue @@ -45,7 +45,7 @@