Skip to content

Commit

Permalink
[core] optimize the logic of printing configuration options
Browse files Browse the repository at this point in the history
  • Loading branch information
chao.tian committed Jun 5, 2023
1 parent 2cf8b83 commit 5deea99
Show file tree
Hide file tree
Showing 9 changed files with 21 additions and 23 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -17,33 +17,25 @@
package com.ververica.cdc.connectors.base.utils;

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.ReadableConfig;
import org.apache.flink.configuration.ConfigurationUtils;

import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

import java.util.Arrays;
import java.util.List;
import java.util.Map;

/** A utility class to print configuration of connectors. */
public class OptionUtils {

private static final Logger LOG = LoggerFactory.getLogger(OptionUtils.class);

private static final List<String> SENSITIVE_OPTIONS = Arrays.asList("password");

private static final String SHADE = "**********";

/** Utility class can not be instantiated. */
private OptionUtils() {}

public static void printOptions(ReadableConfig config, ConfigOption<?>... options) {
public static void printOptions(Map<String, String> config, ConfigOption<?>... options) {
Map<String, String> hideMap = ConfigurationUtils.hideSensitiveValues(config);
for (ConfigOption<?> option : options) {
if (SENSITIVE_OPTIONS.contains(option.key())) {
LOG.info("{} = {}", option.key(), SHADE);
} else {
LOG.info("{} = {}", option.key(), config.get(option));
}
LOG.info("{} = {}", option.key(), hideMap.get(option.key()));
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -18,8 +18,8 @@

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.ConfigOptions;
import org.apache.flink.configuration.Configuration;

import org.junit.jupiter.api.Assertions;
import org.junit.jupiter.api.Test;

import java.util.HashMap;
Expand All @@ -34,10 +34,9 @@ public void testOptionsPrint() {
options.put("host", "localhost:3306");
options.put("user", "tyrantlucifer");
options.put("password", "tyrantlucifer");
Configuration configuration = Configuration.fromMap(options);
ConfigOption<String> host = ConfigOptions.key("host").stringType().noDefaultValue();
ConfigOption<String> user = ConfigOptions.key("user").stringType().noDefaultValue();
ConfigOption<String> password = ConfigOptions.key("password").stringType().noDefaultValue();
OptionUtils.printOptions(configuration, host, user, password);
OptionUtils.printOptions(options, host, user, password);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@
package com.ververica.cdc.connectors.mongodb.table;

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.ReadableConfig;
import org.apache.flink.table.api.config.TableConfigOptions;
import org.apache.flink.table.catalog.ResolvedSchema;
Expand Down Expand Up @@ -99,7 +100,7 @@ public DynamicTableSource createDynamicTableSource(Context context) {
checkPrimaryKey(physicalSchema.getPrimaryKey().get(), "Primary key must be _id field");

OptionUtils.printOptions(
config,
((Configuration) config).toMap(),
HOSTS,
USERNAME,
PASSWORD,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@
package com.ververica.cdc.connectors.mysql.table;

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.ReadableConfig;
import org.apache.flink.table.api.ValidationException;
import org.apache.flink.table.catalog.ResolvedSchema;
Expand Down Expand Up @@ -124,7 +125,7 @@ public DynamicTableSource createDynamicTableSource(Context context) {
}

OptionUtils.printOptions(
config,
((Configuration) config).toMap(),
HOSTNAME,
PORT,
USERNAME,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.ConfigOptions;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.ReadableConfig;
import org.apache.flink.table.catalog.ResolvedSchema;
import org.apache.flink.table.connector.source.DynamicTableSource;
Expand Down Expand Up @@ -190,7 +191,7 @@ public DynamicTableSource createDynamicTableSource(Context context) {
String workingMode = config.get(WORKING_MODE);

OptionUtils.printOptions(
config,
((Configuration) config).toMap(),
SCAN_STARTUP_MODE,
USERNAME,
PASSWORD,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@
package com.ververica.cdc.connectors.oracle.table;

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.ReadableConfig;
import org.apache.flink.table.api.ValidationException;
import org.apache.flink.table.catalog.ResolvedSchema;
Expand Down Expand Up @@ -104,7 +105,7 @@ public DynamicTableSource createDynamicTableSource(Context context) {
}

OptionUtils.printOptions(
config,
((Configuration) config).toMap(),
HOSTNAME,
USERNAME,
PASSWORD,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.ConfigOptions;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.ReadableConfig;
import org.apache.flink.table.catalog.ResolvedSchema;
import org.apache.flink.table.connector.source.DynamicTableSource;
Expand Down Expand Up @@ -138,7 +139,7 @@ public DynamicTableSource createDynamicTableSource(DynamicTableFactory.Context c
}

OptionUtils.printOptions(
config,
((Configuration) config).toMap(),
HOSTNAME,
PORT,
USERNAME,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.ConfigOptions;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.ReadableConfig;
import org.apache.flink.table.api.ValidationException;
import org.apache.flink.table.catalog.ResolvedSchema;
Expand Down Expand Up @@ -118,7 +119,7 @@ public DynamicTableSource createDynamicTableSource(Context context) {
getPhysicalSchema(context.getCatalogTable().getResolvedSchema());

OptionUtils.printOptions(
config,
((Configuration) config).toMap(),
HOSTNAME,
PORT,
USERNAME,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@
package com.ververica.cdc.connectors.tidb.table;

import org.apache.flink.configuration.ConfigOption;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.ReadableConfig;
import org.apache.flink.table.api.ValidationException;
import org.apache.flink.table.catalog.ResolvedSchema;
Expand Down Expand Up @@ -60,7 +61,7 @@ public DynamicTableSource createDynamicTableSource(Context context) {
getPhysicalSchema(context.getCatalogTable().getResolvedSchema());

OptionUtils.printOptions(
config,
((Configuration) config).toMap(),
DATABASE_NAME,
TABLE_NAME,
SCAN_STARTUP_MODE,
Expand Down

0 comments on commit 5deea99

Please sign in to comment.