Skip to content

Commit f0298c6

Browse files
xiuzhu9527tiezhu
authored andcommitted
fix sql urlencode
1 parent 872cb6d commit f0298c6

File tree

3 files changed

+13
-13
lines changed

3 files changed

+13
-13
lines changed

flinkx-core/src/main/java/com/dtstack/flinkx/Main.java

Lines changed: 4 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -71,6 +71,8 @@
7171
import org.slf4j.LoggerFactory;
7272

7373
import java.net.URL;
74+
import java.net.URLDecoder;
75+
import java.nio.charset.StandardCharsets;
7476
import java.util.Arrays;
7577
import java.util.List;
7678
import java.util.Optional;
@@ -93,7 +95,7 @@ public static void main(String[] args) throws Exception {
9395
LOG.info("-------------------------------------------");
9496

9597
Options options = new OptionParser(args).getOptions();
96-
String job = options.getJob();
98+
String job = URLDecoder.decode(options.getJob(), StandardCharsets.UTF_8.name());
9799
Properties confProperties = PropertiesUtil.parseConf(options.getConfProp());
98100
StreamExecutionEnvironment env = EnvFactory.createStreamExecutionEnvironment(options);
99101
StreamTableEnvironment tEnv =
@@ -313,7 +315,7 @@ private static void configStreamExecutionEnvironment(
313315
factoryHelper.setRemotePluginPath(options.getRemoteFlinkxDistDir());
314316
factoryHelper.setPluginLoadMode(options.getPluginLoadMode());
315317
factoryHelper.setEnv(env);
316-
factoryHelper.setOptions(options);
318+
factoryHelper.setExecutionMode(options.getMode());
317319

318320
DirtyConf dirtyConf = DirtyConfUtil.parse(options);
319321
factoryHelper.registerCachedFile(

flinkx-core/src/main/java/com/dtstack/flinkx/util/FactoryHelper.java

Lines changed: 7 additions & 8 deletions
Original file line numberDiff line numberDiff line change
@@ -18,7 +18,6 @@
1818
package com.dtstack.flinkx.util;
1919

2020
import com.dtstack.flinkx.constants.ConstantValue;
21-
import com.dtstack.flinkx.options.Options;
2221

2322
import org.apache.flink.configuration.ConfigOption;
2423
import org.apache.flink.configuration.ConfigOptions;
@@ -61,8 +60,8 @@ public class FactoryHelper {
6160
protected List<URL> classPathSet = new ArrayList<>();
6261
/** shipfile需要的jar的classPath index */
6362
protected int classFileNameIndex = 0;
64-
/** 任务启动参数 */
65-
protected Options options;
63+
/** 任务执行模式 */
64+
protected String executionMode;
6665

6766
public FactoryHelper() {}
6867

@@ -94,7 +93,7 @@ public void registerCachedFile(
9493
this.classFileNameIndex++;
9594
}
9695
}
97-
PluginUtil.setPipelineOptionsToEnvConfig(this.env, urlList, options);
96+
PluginUtil.setPipelineOptionsToEnvConfig(this.env, urlList, executionMode);
9897
} catch (Exception e) {
9998
LOG.warn("can't add jar in {} to cachedFile, e = {}", urlSet, e.getMessage());
10099
}
@@ -116,11 +115,11 @@ public void setEnv(StreamExecutionEnvironment env) {
116115
this.env = env;
117116
}
118117

119-
public Options getOptions() {
120-
return options;
118+
public String getExecutionMode() {
119+
return executionMode;
121120
}
122121

123-
public void setOptions(Options options) {
124-
this.options = options;
122+
public void setExecutionMode(String executionMode) {
123+
this.executionMode = executionMode;
125124
}
126125
}

flinkx-core/src/main/java/com/dtstack/flinkx/util/PluginUtil.java

Lines changed: 2 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -346,7 +346,7 @@ public static void registerPluginUrlToCachedFile(
346346
e);
347347
}
348348

349-
config.setSyncJarList(setPipelineOptionsToEnvConfig(env, urlList, options));
349+
config.setSyncJarList(setPipelineOptionsToEnvConfig(env, urlList, options.getMode()));
350350
}
351351

352352
/**
@@ -358,7 +358,7 @@ public static void registerPluginUrlToCachedFile(
358358
*/
359359
@SuppressWarnings("all")
360360
public static List<String> setPipelineOptionsToEnvConfig(
361-
StreamExecutionEnvironment env, List<String> urlList, Options options) {
361+
StreamExecutionEnvironment env, List<String> urlList, String executionMode) {
362362
try {
363363
Configuration configuration =
364364
(Configuration)
@@ -370,7 +370,6 @@ public static List<String> setPipelineOptionsToEnvConfig(
370370
jarList.addAll(urlList);
371371

372372
List<String> pipelineJars = new ArrayList();
373-
String executionMode = options.getMode();
374373
LOG.info("Flinkx executionMode: " + executionMode);
375374
if (ClusterMode.getByName(executionMode) == ClusterMode.kubernetesApplication) {
376375
for (String jarUrl : jarList) {

0 commit comments

Comments
 (0)