This is an automated email from the ASF dual-hosted git repository.

ic4y pushed a commit to branch dev
in repository https://gitbox.apache.org/repos/asf/incubator-seatunnel.git


The following commit(s) were added to refs/heads/dev by this push:
     new 68f8e950e [Improve][transform] Allow transform to be ignored in job 
files (#3249)
68f8e950e is described below

commit 68f8e950e16f5ecc652126921401edabf5ac6fd2
Author: hailin0 <[email protected]>
AuthorDate: Tue Nov 1 15:15:50 2022 +0800

    [Improve][transform] Allow transform to be ignored in job files (#3249)
---
 .../apache/seatunnel/common/config/TypesafeConfigUtils.java  |  7 +++++++
 .../seatunnel/core/flink/config/FlinkExecutionContext.java   |  4 +++-
 .../seatunnel/core/spark/config/SparkExecutionContext.java   |  4 +++-
 .../core/starter/flink/execution/FlinkExecution.java         |  4 +++-
 .../core/starter/spark/execution/SparkExecution.java         |  5 ++++-
 .../src/test/resources/clickhouse_to_clickhouse.conf         |  5 -----
 .../src/test/resources/json/fake_to_local_file_json.conf     |  3 ---
 .../src/test/resources/json/local_file_json_to_assert.conf   |  3 ---
 .../src/test/resources/orc/fake_to_local_file_orc.conf       |  3 ---
 .../src/test/resources/orc/local_file_orc_to_assert.conf     |  3 ---
 .../test/resources/parquet/fake_to_local_file_parquet.conf   |  3 ---
 .../test/resources/parquet/local_file_parquet_to_assert.conf |  3 ---
 .../src/test/resources/text/fake_to_local_file_text.conf     |  3 ---
 .../src/test/resources/text/local_file_text_to_assert.conf   |  3 ---
 .../src/test/resources/influxdb_source_to_assert.conf        |  6 ------
 .../src/test/resources/jdbc_db2_source_and_sink.conf         |  6 ------
 .../src/test/resources/jdbc_dm_source_and_sink.conf          |  3 ---
 .../src/test/resources/jdbc_gbase8a_source_to_assert.conf    |  6 ------
 .../src/test/resources/jdbc_oracle_source_to_sink.conf       |  6 ------
 .../src/test/resources/jdbc_starrocks_source_to_sink.conf    |  4 ----
 .../src/test/resources/redis-to-redis.conf                   |  3 ---
 .../src/test/resources/fakesource_to_console.conf            |  3 ---
 .../src/test/resources/fake/fakesource_to_console.conf       |  9 ---------
 .../src/test/resources/fake/fakesource_to_console.conf       | 12 ------------
 .../apache/seatunnel/engine/core/parse/JobConfigParser.java  |  3 ++-
 25 files changed, 22 insertions(+), 92 deletions(-)

diff --git 
a/seatunnel-common/src/main/java/org/apache/seatunnel/common/config/TypesafeConfigUtils.java
 
b/seatunnel-common/src/main/java/org/apache/seatunnel/common/config/TypesafeConfigUtils.java
index 777244b1e..02cc58d07 100644
--- 
a/seatunnel-common/src/main/java/org/apache/seatunnel/common/config/TypesafeConfigUtils.java
+++ 
b/seatunnel-common/src/main/java/org/apache/seatunnel/common/config/TypesafeConfigUtils.java
@@ -24,6 +24,7 @@ import 
org.apache.seatunnel.shade.com.typesafe.config.ConfigValue;
 import lombok.NonNull;
 
 import java.util.LinkedHashMap;
+import java.util.List;
 import java.util.Map;
 
 public final class TypesafeConfigUtils {
@@ -110,4 +111,10 @@ public final class TypesafeConfigUtils {
         }
         throw new RuntimeException("Unsupported config type, configKey: " + 
configKey);
     }
+
+    public static List<? extends Config> getConfigList(Config config,
+                                                       String configKey,
+                                                       @NonNull List<? extends 
Config> defaultValue) {
+        return config.hasPath(configKey) ? config.getConfigList(configKey) : 
defaultValue;
+    }
 }
diff --git 
a/seatunnel-core/seatunnel-core-flink/src/main/java/org/apache/seatunnel/core/flink/config/FlinkExecutionContext.java
 
b/seatunnel-core/seatunnel-core-flink/src/main/java/org/apache/seatunnel/core/flink/config/FlinkExecutionContext.java
index 2acb83699..a9da690cb 100644
--- 
a/seatunnel-core/seatunnel-core-flink/src/main/java/org/apache/seatunnel/core/flink/config/FlinkExecutionContext.java
+++ 
b/seatunnel-core/seatunnel-core-flink/src/main/java/org/apache/seatunnel/core/flink/config/FlinkExecutionContext.java
@@ -20,6 +20,7 @@ package org.apache.seatunnel.core.flink.config;
 import org.apache.seatunnel.apis.base.api.BaseSink;
 import org.apache.seatunnel.apis.base.api.BaseSource;
 import org.apache.seatunnel.apis.base.api.BaseTransform;
+import org.apache.seatunnel.common.config.TypesafeConfigUtils;
 import org.apache.seatunnel.common.constants.PluginType;
 import org.apache.seatunnel.core.base.config.AbstractExecutionContext;
 import org.apache.seatunnel.core.base.config.EngineType;
@@ -33,6 +34,7 @@ import org.apache.seatunnel.shade.com.typesafe.config.Config;
 
 import java.net.URL;
 import java.util.ArrayList;
+import java.util.Collections;
 import java.util.HashSet;
 import java.util.List;
 import java.util.Set;
@@ -75,7 +77,7 @@ public class FlinkExecutionContext extends 
AbstractExecutionContext<FlinkEnviron
     public List<BaseTransform<FlinkEnvironment>> getTransforms() {
         final String pluginType = PluginType.TRANSFORM.getType();
         final String engineType = EngineType.FLINK.getEngine();
-        final List<? extends Config> configList = 
getRootConfig().getConfigList(pluginType);
+        final List<? extends Config> configList = 
TypesafeConfigUtils.getConfigList(getRootConfig(), pluginType, 
Collections.emptyList());
         return configList.stream()
             .map(pluginConfig -> {
                 PluginIdentifier pluginIdentifier = 
PluginIdentifier.of(engineType, pluginType, 
pluginConfig.getString("plugin_name"));
diff --git 
a/seatunnel-core/seatunnel-core-spark/src/main/java/org/apache/seatunnel/core/spark/config/SparkExecutionContext.java
 
b/seatunnel-core/seatunnel-core-spark/src/main/java/org/apache/seatunnel/core/spark/config/SparkExecutionContext.java
index 07854b83c..0574f84c8 100644
--- 
a/seatunnel-core/seatunnel-core-spark/src/main/java/org/apache/seatunnel/core/spark/config/SparkExecutionContext.java
+++ 
b/seatunnel-core/seatunnel-core-spark/src/main/java/org/apache/seatunnel/core/spark/config/SparkExecutionContext.java
@@ -20,6 +20,7 @@ package org.apache.seatunnel.core.spark.config;
 import org.apache.seatunnel.apis.base.api.BaseSink;
 import org.apache.seatunnel.apis.base.api.BaseSource;
 import org.apache.seatunnel.apis.base.api.BaseTransform;
+import org.apache.seatunnel.common.config.TypesafeConfigUtils;
 import org.apache.seatunnel.common.constants.PluginType;
 import org.apache.seatunnel.core.base.config.AbstractExecutionContext;
 import org.apache.seatunnel.core.base.config.EngineType;
@@ -33,6 +34,7 @@ import org.apache.seatunnel.shade.com.typesafe.config.Config;
 
 import java.net.URL;
 import java.util.ArrayList;
+import java.util.Collections;
 import java.util.HashSet;
 import java.util.List;
 import java.util.Set;
@@ -74,7 +76,7 @@ public class SparkExecutionContext extends 
AbstractExecutionContext<SparkEnviron
     public List<BaseTransform<SparkEnvironment>> getTransforms() {
         final String pluginType = PluginType.TRANSFORM.getType();
         final String engineType = EngineType.SPARK.getEngine();
-        final List<? extends Config> configList = 
getRootConfig().getConfigList(pluginType);
+        final List<? extends Config> configList = 
TypesafeConfigUtils.getConfigList(getRootConfig(), pluginType, 
Collections.emptyList());
         return configList.stream()
             .map(pluginConfig -> {
                 PluginIdentifier pluginIdentifier = 
PluginIdentifier.of(engineType, pluginType, 
pluginConfig.getString("plugin_name"));
diff --git 
a/seatunnel-core/seatunnel-flink-starter/src/main/java/org/apache/seatunnel/core/starter/flink/execution/FlinkExecution.java
 
b/seatunnel-core/seatunnel-flink-starter/src/main/java/org/apache/seatunnel/core/starter/flink/execution/FlinkExecution.java
index ddf346a4c..1dbcf04f9 100644
--- 
a/seatunnel-core/seatunnel-flink-starter/src/main/java/org/apache/seatunnel/core/starter/flink/execution/FlinkExecution.java
+++ 
b/seatunnel-core/seatunnel-flink-starter/src/main/java/org/apache/seatunnel/core/starter/flink/execution/FlinkExecution.java
@@ -20,6 +20,7 @@ package org.apache.seatunnel.core.starter.flink.execution;
 import org.apache.seatunnel.api.common.JobContext;
 import org.apache.seatunnel.common.Constants;
 import org.apache.seatunnel.common.config.Common;
+import org.apache.seatunnel.common.config.TypesafeConfigUtils;
 import org.apache.seatunnel.common.utils.SeaTunnelException;
 import org.apache.seatunnel.core.starter.exception.TaskExecuteException;
 import org.apache.seatunnel.core.starter.execution.TaskExecution;
@@ -72,7 +73,8 @@ public class FlinkExecution implements TaskExecution {
         jobContext.setJobMode(FlinkEnvironmentFactory.getJobMode(config));
 
         this.sourcePluginExecuteProcessor = new 
SourceExecuteProcessor(jarPaths, config.getConfigList(Constants.SOURCE), 
jobContext);
-        this.transformPluginExecuteProcessor = new 
TransformExecuteProcessor(jarPaths, config.getConfigList(Constants.TRANSFORM), 
jobContext);
+        this.transformPluginExecuteProcessor = new 
TransformExecuteProcessor(jarPaths,
+            TypesafeConfigUtils.getConfigList(config, Constants.TRANSFORM, 
Collections.emptyList()), jobContext);
         this.sinkPluginExecuteProcessor = new SinkExecuteProcessor(jarPaths, 
config.getConfigList(Constants.SINK), jobContext);
 
         this.flinkEnvironment = new 
FlinkEnvironmentFactory(this.registerPlugin(config, jarPaths)).getEnvironment();
diff --git 
a/seatunnel-core/seatunnel-spark-starter/src/main/java/org/apache/seatunnel/core/starter/spark/execution/SparkExecution.java
 
b/seatunnel-core/seatunnel-spark-starter/src/main/java/org/apache/seatunnel/core/starter/spark/execution/SparkExecution.java
index ed2e60e91..9980e77f1 100644
--- 
a/seatunnel-core/seatunnel-spark-starter/src/main/java/org/apache/seatunnel/core/starter/spark/execution/SparkExecution.java
+++ 
b/seatunnel-core/seatunnel-spark-starter/src/main/java/org/apache/seatunnel/core/starter/spark/execution/SparkExecution.java
@@ -19,6 +19,7 @@ package org.apache.seatunnel.core.starter.spark.execution;
 
 import org.apache.seatunnel.api.common.JobContext;
 import org.apache.seatunnel.common.Constants;
+import org.apache.seatunnel.common.config.TypesafeConfigUtils;
 import org.apache.seatunnel.core.starter.exception.TaskExecuteException;
 import org.apache.seatunnel.core.starter.spark.config.SparkEnvironmentFactory;
 import org.apache.seatunnel.spark.SparkEnvironment;
@@ -30,6 +31,7 @@ import org.apache.spark.sql.Dataset;
 import org.apache.spark.sql.Row;
 
 import java.util.ArrayList;
+import java.util.Collections;
 import java.util.List;
 
 @Slf4j
@@ -45,7 +47,8 @@ public class SparkExecution {
         JobContext jobContext = new JobContext();
         jobContext.setJobMode(sparkEnvironment.getJobMode());
         this.sourcePluginExecuteProcessor = new 
SourceExecuteProcessor(sparkEnvironment, jobContext, 
config.getConfigList(Constants.SOURCE));
-        this.transformPluginExecuteProcessor = new 
TransformExecuteProcessor(sparkEnvironment, jobContext, 
config.getConfigList(Constants.TRANSFORM));
+        this.transformPluginExecuteProcessor = new 
TransformExecuteProcessor(sparkEnvironment, jobContext,
+            TypesafeConfigUtils.getConfigList(config, Constants.TRANSFORM, 
Collections.emptyList()));
         this.sinkPluginExecuteProcessor = new 
SinkExecuteProcessor(sparkEnvironment, jobContext, 
config.getConfigList(Constants.SINK));
     }
 
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-clickhouse-e2e/src/test/resources/clickhouse_to_clickhouse.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-clickhouse-e2e/src/test/resources/clickhouse_to_clickhouse.conf
index ed27371b8..ec4821c77 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-clickhouse-e2e/src/test/resources/clickhouse_to_clickhouse.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-clickhouse-e2e/src/test/resources/clickhouse_to_clickhouse.conf
@@ -38,11 +38,6 @@ source {
   # please go to 
https://seatunnel.apache.org/docs/connector-v2/source/ClickhouseSource
 }
 
-transform {
-  # If you would like to get more information about how to configure seatunnel 
and see full list of transform plugins,
-  # please go to https://seatunnel.apache.org/docs/transform/sql
-}
-
 sink {
   Clickhouse {
     host = "clickhouse:8123"
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/json/fake_to_local_file_json.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/json/fake_to_local_file_json.conf
index 0dc2cc054..9d48af1cf 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/json/fake_to_local_file_json.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/json/fake_to_local_file_json.conf
@@ -65,9 +65,6 @@ source {
   }
 }
 
-transform {
-}
-
 sink {
   LocalFile {
     path = "/tmp/seatunnel/json"
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/json/local_file_json_to_assert.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/json/local_file_json_to_assert.conf
index 901f90984..ba94dce76 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/json/local_file_json_to_assert.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/json/local_file_json_to_assert.conf
@@ -67,9 +67,6 @@ source {
   }
 }
 
-transform {
-}
-
 sink {
   Assert {
     rules {
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/orc/fake_to_local_file_orc.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/orc/fake_to_local_file_orc.conf
index 647b7a17a..cbbc227b3 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/orc/fake_to_local_file_orc.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/orc/fake_to_local_file_orc.conf
@@ -65,9 +65,6 @@ source {
   }
 }
 
-transform {
-}
-
 sink {
   LocalFile {
     path = "/tmp/seatunnel/orc"
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/orc/local_file_orc_to_assert.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/orc/local_file_orc_to_assert.conf
index eed920ab3..d99fef31a 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/orc/local_file_orc_to_assert.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/orc/local_file_orc_to_assert.conf
@@ -33,9 +33,6 @@ source {
   }
 }
 
-transform {
-}
-
 sink {
   Assert {
     rules {
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/parquet/fake_to_local_file_parquet.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/parquet/fake_to_local_file_parquet.conf
index 531dd8211..454e27f41 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/parquet/fake_to_local_file_parquet.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/parquet/fake_to_local_file_parquet.conf
@@ -65,9 +65,6 @@ source {
   }
 }
 
-transform {
-}
-
 sink {
   LocalFile {
     path = "/tmp/seatunnel/parquet"
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/parquet/local_file_parquet_to_assert.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/parquet/local_file_parquet_to_assert.conf
index eed920ab3..d99fef31a 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/parquet/local_file_parquet_to_assert.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/parquet/local_file_parquet_to_assert.conf
@@ -33,9 +33,6 @@ source {
   }
 }
 
-transform {
-}
-
 sink {
   Assert {
     rules {
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/text/fake_to_local_file_text.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/text/fake_to_local_file_text.conf
index 7bbe3cdb5..54227c8ee 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/text/fake_to_local_file_text.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/text/fake_to_local_file_text.conf
@@ -65,9 +65,6 @@ source {
   }
 }
 
-transform {
-}
-
 sink {
   LocalFile {
     path = "/tmp/seatunnel/text"
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/text/local_file_text_to_assert.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/text/local_file_text_to_assert.conf
index 0f16a37b9..11c0a5710 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/text/local_file_text_to_assert.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-file-local-e2e/src/test/resources/text/local_file_text_to_assert.conf
@@ -67,9 +67,6 @@ source {
   }
 }
 
-transform {
-}
-
 sink {
   Assert {
     rules {
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-influxdb-e2e/src/test/resources/influxdb_source_to_assert.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-influxdb-e2e/src/test/resources/influxdb_source_to_assert.conf
index 2a2f1192c..ea0e6e177 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-influxdb-e2e/src/test/resources/influxdb_source_to_assert.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-influxdb-e2e/src/test/resources/influxdb_source_to_assert.conf
@@ -53,12 +53,6 @@ source {
     }
 }
 
-transform {
-
-  # If you would like to get more information about how to configure seatunnel 
and see full list of transform plugins,
-  # please go to https://seatunnel.apache.org/docs/transform/sql
-}
-
 sink {
      Assert {
          rules =
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_db2_source_and_sink.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_db2_source_and_sink.conf
index cc2e64cfd..0c8ff8e4b 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_db2_source_and_sink.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_db2_source_and_sink.conf
@@ -56,12 +56,6 @@ source {
   # please go to https://seatunnel.apache.org/docs/connector-v2/source/Jdbc
 }
 
-transform {
-
-  # If you would like to get more information about how to configure seatunnel 
and see full list of transform plugins,
-  # please go to https://seatunnel.apache.org/docs/transform/sql
-}
-
 sink {
   Jdbc {
     driver = com.ibm.db2.jcc.DB2Driver
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_dm_source_and_sink.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_dm_source_and_sink.conf
index a536be5e0..7a3ac4334 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_dm_source_and_sink.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_dm_source_and_sink.conf
@@ -32,9 +32,6 @@ source {
 
 }
 
-transform {
-}
-
 sink {
   Jdbc {
     url = "jdbc:dm://e2e_dmdb:5236"
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_gbase8a_source_to_assert.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_gbase8a_source_to_assert.conf
index 4b4c5ad62..aff8f0096 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_gbase8a_source_to_assert.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_gbase8a_source_to_assert.conf
@@ -40,12 +40,6 @@ source {
   # please go to 
https://seatunnel.apache.org/docs/connector-v2/source/FakeSource
 }
 
-transform {
-
-  # If you would like to get more information about how to configure seatunnel 
and see full list of transform plugins,
-  # please go to https://seatunnel.apache.org/docs/transform/sql
-}
-
 sink {
   Assert {
     rules =
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_oracle_source_to_sink.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_oracle_source_to_sink.conf
index 11cc07e90..c67a4d2fe 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_oracle_source_to_sink.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_oracle_source_to_sink.conf
@@ -40,12 +40,6 @@ source {
   # please go to 
https://seatunnel.apache.org/docs/connector-v2/source/FakeSource
 }
 
-transform {
-
-  # If you would like to get more information about how to configure seatunnel 
and see full list of transform plugins,
-  # please go to https://seatunnel.apache.org/docs/transform/sql
-}
-
 sink {
   Jdbc {
     driver = oracle.jdbc.driver.OracleDriver
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_starrocks_source_to_sink.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_starrocks_source_to_sink.conf
index 056cf47db..54886d059 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_starrocks_source_to_sink.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-jdbc-e2e/src/test/resources/jdbc_starrocks_source_to_sink.conf
@@ -30,10 +30,6 @@ source {
   }
 }
 
-transform {
-
-}
-
 sink {
   Jdbc {
     driver = com.mysql.cj.jdbc.Driver
diff --git 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-redis-e2e/src/test/resources/redis-to-redis.conf
 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-redis-e2e/src/test/resources/redis-to-redis.conf
index 1c15644f1..bdb29e187 100644
--- 
a/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-redis-e2e/src/test/resources/redis-to-redis.conf
+++ 
b/seatunnel-e2e/seatunnel-connector-v2-e2e/connector-redis-e2e/src/test/resources/redis-to-redis.conf
@@ -37,9 +37,6 @@ source {
     }
 }
 
-transform {
-}
-
 sink {
     Redis {
         host = "redis-e2e"
diff --git 
a/seatunnel-e2e/seatunnel-engine-e2e/connector-console-seatunnel-e2e/src/test/resources/fakesource_to_console.conf
 
b/seatunnel-e2e/seatunnel-engine-e2e/connector-console-seatunnel-e2e/src/test/resources/fakesource_to_console.conf
index 3b124fe8b..ffc827d8f 100644
--- 
a/seatunnel-e2e/seatunnel-engine-e2e/connector-console-seatunnel-e2e/src/test/resources/fakesource_to_console.conf
+++ 
b/seatunnel-e2e/seatunnel-engine-e2e/connector-console-seatunnel-e2e/src/test/resources/fakesource_to_console.conf
@@ -36,9 +36,6 @@ source {
     }
 }
 
-transform {
-}
-
 sink {
   console {
   }
diff --git 
a/seatunnel-e2e/seatunnel-flink-e2e/seatunnel-connector-flink-fake-e2e/src/test/resources/fake/fakesource_to_console.conf
 
b/seatunnel-e2e/seatunnel-flink-e2e/seatunnel-connector-flink-fake-e2e/src/test/resources/fake/fakesource_to_console.conf
index ba0b57305..23c6337fb 100644
--- 
a/seatunnel-e2e/seatunnel-flink-e2e/seatunnel-connector-flink-fake-e2e/src/test/resources/fake/fakesource_to_console.conf
+++ 
b/seatunnel-e2e/seatunnel-flink-e2e/seatunnel-connector-flink-fake-e2e/src/test/resources/fake/fakesource_to_console.conf
@@ -36,15 +36,6 @@ source {
   # please go to 
https://seatunnel.apache.org/docs/flink/configuration/source-plugins/Fake
 }
 
-transform {
-    sql {
-      sql = "select name,age from fake"
-    }
-
-  # If you would like to get more information about how to configure seatunnel 
and see full list of transform plugins,
-  # please go to 
https://seatunnel.apache.org/docs/flink/configuration/transform-plugins/Sql
-}
-
 sink {
   ConsoleSink {
   }
diff --git 
a/seatunnel-e2e/seatunnel-spark-e2e/seatunnel-connector-spark-fake-e2e/src/test/resources/fake/fakesource_to_console.conf
 
b/seatunnel-e2e/seatunnel-spark-e2e/seatunnel-connector-spark-fake-e2e/src/test/resources/fake/fakesource_to_console.conf
index dd7ca72c0..9ba0c0dd4 100644
--- 
a/seatunnel-e2e/seatunnel-spark-e2e/seatunnel-connector-spark-fake-e2e/src/test/resources/fake/fakesource_to_console.conf
+++ 
b/seatunnel-e2e/seatunnel-spark-e2e/seatunnel-connector-spark-fake-e2e/src/test/resources/fake/fakesource_to_console.conf
@@ -45,18 +45,6 @@ source {
   # please go to 
https://seatunnel.apache.org/docs/spark/configuration/source-plugins/Fake
 }
 
-transform {
-  # split data by specific delimiter
-
-  # you can also use other transform plugins, such as sql
-  # sql {
-  #   sql = "select * from accesslog where request_time > 1000"
-  # }
-
-  # If you would like to get more information about how to configure seatunnel 
and see full list of transform plugins,
-  # please go to 
https://seatunnel.apache.org/docs/spark/configuration/transform-plugins/Split
-}
-
 sink {
   # choose stdout output plugin to output data to console
   Console {}
diff --git 
a/seatunnel-engine/seatunnel-engine-core/src/main/java/org/apache/seatunnel/engine/core/parse/JobConfigParser.java
 
b/seatunnel-engine/seatunnel-engine-core/src/main/java/org/apache/seatunnel/engine/core/parse/JobConfigParser.java
index 92041823b..4abb3351a 100644
--- 
a/seatunnel-engine/seatunnel-engine-core/src/main/java/org/apache/seatunnel/engine/core/parse/JobConfigParser.java
+++ 
b/seatunnel-engine/seatunnel-engine-core/src/main/java/org/apache/seatunnel/engine/core/parse/JobConfigParser.java
@@ -25,6 +25,7 @@ import org.apache.seatunnel.api.table.type.SeaTunnelRowType;
 import org.apache.seatunnel.api.transform.PartitionSeaTunnelTransform;
 import org.apache.seatunnel.api.transform.SeaTunnelTransform;
 import org.apache.seatunnel.apis.base.plugin.Plugin;
+import org.apache.seatunnel.common.config.TypesafeConfigUtils;
 import org.apache.seatunnel.common.constants.CollectionConstants;
 import org.apache.seatunnel.common.constants.JobMode;
 import org.apache.seatunnel.core.starter.config.ConfigBuilder;
@@ -104,7 +105,7 @@ public class JobConfigParser {
 
     public ImmutablePair<List<Action>, Set<URL>> parse() {
         List<? extends Config> sinkConfigs = 
seaTunnelJobConfig.getConfigList("sink");
-        List<? extends Config> transformConfigs = 
seaTunnelJobConfig.getConfigList("transform");
+        List<? extends Config> transformConfigs = 
TypesafeConfigUtils.getConfigList(seaTunnelJobConfig, "transform", 
Collections.emptyList());
         List<? extends Config> sourceConfigs = 
seaTunnelJobConfig.getConfigList("source");
 
         if (CollectionUtils.isEmpty(sinkConfigs) || 
CollectionUtils.isEmpty(sourceConfigs)) {

Reply via email to