wuchong commented on a change in pull request #12176: URL: https://github.com/apache/flink/pull/12176#discussion_r426133850
########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSource.java ########## @@ -0,0 +1,162 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.api.java.typeutils.RowTypeInfo; +import org.apache.flink.connector.jdbc.dialect.JdbcDialect; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.connector.jdbc.split.JdbcNumericBetweenParametersProvider; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.ChangelogMode; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.connector.source.InputFormatProvider; +import org.apache.flink.table.connector.source.LookupTableSource; +import org.apache.flink.table.connector.source.ScanTableSource; +import org.apache.flink.table.connector.source.TableFunctionProvider; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.types.logical.LogicalType; +import org.apache.flink.table.types.logical.RowType; + +import java.util.Arrays; +import java.util.Objects; + +import static org.apache.flink.table.types.utils.TypeConversions.fromDataTypeToLegacyInfo; + +/** + * A {@link DynamicTableSource} for JDBC. + */ +@Internal +public class JdbcDynamicTableSource implements ScanTableSource, LookupTableSource { + + private static final String name = "JdbcTableSource"; + private final JdbcOptions options; + private final JdbcReadOptions readOptions; + private final JdbcLookupOptions lookupOptions; + private final TableSchema schema; + private final int[] selectFields; + + public JdbcDynamicTableSource( + JdbcOptions options, + JdbcReadOptions readOptions, + JdbcLookupOptions lookupOptions, + TableSchema schema, + int[] selectFields) { + this.options = options; + this.readOptions = readOptions; + this.lookupOptions = lookupOptions; + this.schema = schema; + this.selectFields = selectFields; + } + + @Override + public LookupRuntimeProvider getLookupRuntimeProvider(LookupTableSource.Context context) { + // JDBC only support non-nested look up keys + String[] keyNames = new String[context.getKeys().length]; + for (int i = 0; i < keyNames.length; i++) { + int index = context.getKeys()[i][0]; + keyNames[i] = schema.getFieldNames()[index]; + } + return TableFunctionProvider.of(JdbcDynamicLookupFunction.builder() + .setFieldNames(schema.getFieldNames()) + .setFieldTypes(schema.getFieldDataTypes()) + .setKeyNames(keyNames) + .setOptions(options) + .setLookupOptions(lookupOptions) + .build()); + } + + @Override + public ScanRuntimeProvider getScanRuntimeProvider(ScanTableSource.Context runtimeProviderContext) { + final DataType rowDataType = schema.toPhysicalRowDataType(); + final RowTypeInfo rowTypeInfo = (RowTypeInfo) fromDataTypeToLegacyInfo(rowDataType); + final JdbcDynamicInputFormat.Builder builder = JdbcDynamicInputFormat.builder() + .setDrivername(options.getDriverName()) + .setDBUrl(options.getDbURL()) + .setUsername(options.getUsername().orElse(null)) + .setPassword(options.getPassword().orElse(null)) + .setRowTypeInfo(new RowTypeInfo(rowTypeInfo.getFieldTypes(), rowTypeInfo.getFieldNames())); + + if (readOptions.getFetchSize() != 0) { + builder.setFetchSize(readOptions.getFetchSize()); + } + final JdbcDialect dialect = options.getDialect(); + String query = dialect.getSelectFromStatement( + options.getTableName(), rowTypeInfo.getFieldNames(), new String[0]); + if (readOptions.getPartitionColumnName().isPresent()) { + long lowerBound = readOptions.getPartitionLowerBound().get(); + long upperBound = readOptions.getPartitionUpperBound().get(); + int numPartitions = readOptions.getNumPartitions().get(); + builder.setParametersProvider( + new JdbcNumericBetweenParametersProvider(lowerBound, upperBound).ofBatchNum(numPartitions)); + query += " WHERE " + + dialect.quoteIdentifier(readOptions.getPartitionColumnName().get()) + + " BETWEEN ? AND ?"; + } + builder.setQuery(query); + final RowType rowType = RowType.of( + Arrays.stream(schema.getFieldDataTypes()) + .map(DataType::getLogicalType) + .toArray(LogicalType[]::new), + schema.getFieldNames()); + builder.setRowConverter(dialect.getInputConverter(rowType)); + + return InputFormatProvider.of(builder.build()); + } + + @Override + public ChangelogMode getChangelogMode() { + return ChangelogMode.insertOnly(); + } + + @Override + public DynamicTableSource copy() { + return new JdbcDynamicTableSource(options, readOptions, lookupOptions, schema, selectFields); + } + + @Override + public String asSummaryString() { + return name; Review comment: ditto. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSource.java ########## @@ -0,0 +1,162 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.api.java.typeutils.RowTypeInfo; +import org.apache.flink.connector.jdbc.dialect.JdbcDialect; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.connector.jdbc.split.JdbcNumericBetweenParametersProvider; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.ChangelogMode; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.connector.source.InputFormatProvider; +import org.apache.flink.table.connector.source.LookupTableSource; +import org.apache.flink.table.connector.source.ScanTableSource; +import org.apache.flink.table.connector.source.TableFunctionProvider; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.types.logical.LogicalType; +import org.apache.flink.table.types.logical.RowType; + +import java.util.Arrays; +import java.util.Objects; + +import static org.apache.flink.table.types.utils.TypeConversions.fromDataTypeToLegacyInfo; + +/** + * A {@link DynamicTableSource} for JDBC. + */ +@Internal +public class JdbcDynamicTableSource implements ScanTableSource, LookupTableSource { + + private static final String name = "JdbcTableSource"; + private final JdbcOptions options; + private final JdbcReadOptions readOptions; + private final JdbcLookupOptions lookupOptions; + private final TableSchema schema; + private final int[] selectFields; + + public JdbcDynamicTableSource( + JdbcOptions options, + JdbcReadOptions readOptions, + JdbcLookupOptions lookupOptions, + TableSchema schema, + int[] selectFields) { + this.options = options; + this.readOptions = readOptions; + this.lookupOptions = lookupOptions; + this.schema = schema; + this.selectFields = selectFields; + } + + @Override + public LookupRuntimeProvider getLookupRuntimeProvider(LookupTableSource.Context context) { + // JDBC only support non-nested look up keys Review comment: Add a check that the length of inner array of `context#getKeys()` should be 1. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSourceSinkFactory.java ########## @@ -0,0 +1,283 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.configuration.ConfigOption; +import org.apache.flink.configuration.ConfigOptions; +import org.apache.flink.configuration.ReadableConfig; +import org.apache.flink.connector.jdbc.JdbcExecutionOptions; +import org.apache.flink.connector.jdbc.dialect.JdbcDialects; +import org.apache.flink.connector.jdbc.internal.options.JdbcDmlOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.sink.DynamicTableSink; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.factories.DynamicTableSinkFactory; +import org.apache.flink.table.factories.DynamicTableSourceFactory; +import org.apache.flink.table.factories.FactoryUtil; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.utils.TableSchemaUtils; + +import java.time.Duration; +import java.util.HashSet; +import java.util.Optional; +import java.util.Set; + +/** + * Factory for creating configured instances of {@link JdbcDynamicTableSource} + * and {@link JdbcDynamicTableSink}. + */ +@Internal +public class JdbcDynamicTableSourceSinkFactory implements DynamicTableSourceFactory, DynamicTableSinkFactory { + + public static final ConfigOption<String> IDENTIFIER = ConfigOptions + .key("connector") + .stringType() + .defaultValue("jdbc") + .withDescription("-- required: specify this table type is jdbc."); Review comment: Remove the `-- ` prefix, and upper case the first character. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSink.java ########## @@ -0,0 +1,113 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.api.common.typeinfo.TypeInformation; +import org.apache.flink.connector.jdbc.JdbcExecutionOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcDmlOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.table.connector.ChangelogMode; +import org.apache.flink.table.connector.sink.DynamicTableSink; +import org.apache.flink.table.connector.sink.OutputFormatProvider; +import org.apache.flink.table.data.RowData; +import org.apache.flink.table.types.DataType; +import org.apache.flink.types.RowKind; + +import java.util.Arrays; +import java.util.Objects; + +import static org.apache.flink.connector.jdbc.table.JdbcDynamicOutputFormat.DynamicOutputFormatBuilder; + +/** + * A {@link DynamicTableSink} for JDBC. + */ +@Internal +public class JdbcDynamicTableSink implements DynamicTableSink { + + private static final String name = "JdbcTableSink"; + private final JdbcOptions jdbcOptions; + private final JdbcExecutionOptions executionOptions; + private final JdbcDmlOptions dmlOptions; + private final DataType rowDataType; + private final DataType[] fieldTypes; Review comment: You can keep the physical `TableSchema` in the sink, it's easy to get the row DataType and field DataTypes. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSourceSinkFactory.java ########## @@ -0,0 +1,283 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.configuration.ConfigOption; +import org.apache.flink.configuration.ConfigOptions; +import org.apache.flink.configuration.ReadableConfig; +import org.apache.flink.connector.jdbc.JdbcExecutionOptions; +import org.apache.flink.connector.jdbc.dialect.JdbcDialects; +import org.apache.flink.connector.jdbc.internal.options.JdbcDmlOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.sink.DynamicTableSink; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.factories.DynamicTableSinkFactory; +import org.apache.flink.table.factories.DynamicTableSourceFactory; +import org.apache.flink.table.factories.FactoryUtil; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.utils.TableSchemaUtils; + +import java.time.Duration; +import java.util.HashSet; +import java.util.Optional; +import java.util.Set; + +/** + * Factory for creating configured instances of {@link JdbcDynamicTableSource} + * and {@link JdbcDynamicTableSink}. + */ +@Internal +public class JdbcDynamicTableSourceSinkFactory implements DynamicTableSourceFactory, DynamicTableSinkFactory { + + public static final ConfigOption<String> IDENTIFIER = ConfigOptions + .key("connector") + .stringType() + .defaultValue("jdbc") + .withDescription("-- required: specify this table type is jdbc."); + public static final ConfigOption<String> URL = ConfigOptions + .key("url") + .stringType() + .noDefaultValue() + .withDescription("-- required: the jdbc database url."); + public static final ConfigOption<String> TABLE = ConfigOptions + .key("table") + .stringType() + .noDefaultValue() + .withDescription("-- required: the jdbc table name."); + public static final ConfigOption<String> USERNAME = ConfigOptions + .key("username") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the jdbc user name."); + public static final ConfigOption<String> PASSWORD = ConfigOptions + .key("password") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the jdbc password."); + private static final ConfigOption<String> DRIVER = ConfigOptions + .key("driver") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the class name of the JDBC driver to use to connect to this URL. " + + "If not set, it will automatically be derived from the URL."); + + // read config options + private static final ConfigOption<String> READ_PARTITION_COLUMN = ConfigOptions + .key("read.partition.column") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the column name used for partitioning the input."); + private static final ConfigOption<Integer> READ_PARTITION_NUM = ConfigOptions + .key("read.partition.num") + .intType() + .noDefaultValue() + .withDescription("-- optional: the number of partitions."); + private static final ConfigOption<Long> READ_PARTITION_LOWER_BOUND = ConfigOptions + .key("read.partition.lower-bound") + .longType() + .noDefaultValue() + .withDescription("-- optional: the smallest value of the first partition."); + private static final ConfigOption<Long> READ_PARTITION_UPPER_BOUND = ConfigOptions + .key("read.partition.upper-bound") + .longType() + .noDefaultValue() + .withDescription("-- optional: the largest value of the last partition."); + private static final ConfigOption<Integer> READ_FETCH_SIZE = ConfigOptions + .key("read.fetch-size") + .intType() + .defaultValue(0) + .withDescription("-- optional, Gives the reader a hint as to the number of rows that should be fetched, from" + + " the database when reading per round trip. If the value specified is zero, then the hint is ignored. The" + + " default value is zero."); + + // look up config options + private static final ConfigOption<Long> LOOKUP_CACHE_MAX_ROWS = ConfigOptions + .key("lookup.cache.max-rows") + .longType() + .defaultValue(-1L) + .withDescription("-- optional, max number of rows of lookup cache, over this value, the oldest rows will " + + "be eliminated. \"cache.max-rows\" and \"cache.ttl\" options must all be specified if any of them is " + + "specified. Cache is not enabled as default."); + private static final ConfigOption<Duration> LOOKUP_CACHE_TTL = ConfigOptions + .key("lookup.cache.ttl") + .durationType() + .defaultValue(Duration.ofSeconds(10)) + .withDescription("-- optional, the cache time to live."); + private static final ConfigOption<Integer> LOOKUP_MAX_RETRIES = ConfigOptions + .key("lookup.max-retries") + .intType() + .defaultValue(3) + .withDescription("-- optional, max retry times if lookup database failed."); + + // write config options + private static final ConfigOption<Integer> WRITE_FLUSH_MAX_ROWS = ConfigOptions + .key("write.flush.max-rows") + .intType() + .defaultValue(5000) + .withDescription("-- optional, flush max size (includes all append, upsert and delete records), over this number" + + " of records, will flush data. The default value is 5000."); + private static final ConfigOption<Long> WRITE_FLUSH_INTERVAL = ConfigOptions + .key("write.flush.interval") + .longType() + .defaultValue(0L) + .withDescription("-- optional, flush interval mills, over this time, asynchronous threads will flush data. The " + + "default value is 0, which means no asynchronous flush thread will be scheduled."); + private static final ConfigOption<Integer> WRITE_MAX_RETRIES = ConfigOptions + .key("write.max-retries") + .intType() + .defaultValue(3) + .withDescription("-- optional, max retry times if writing records to database failed."); + + @Override + public DynamicTableSink createDynamicTableSink(Context context) { + final FactoryUtil.TableFactoryHelper helper = FactoryUtil.createTableFactoryHelper(this, context); + helper.validate(); + final JdbcOptions jdbcOptions = getJdbcOptions(helper.getOptions()); + final DataType rowDataType = context.getCatalogTable().getSchema().toPhysicalRowDataType(); + final TableSchema formatSchema = TableSchemaUtils.getPhysicalSchema(context.getCatalogTable().getSchema()); + final DataType[] fieldDataTypes = formatSchema.getFieldDataTypes(); + + return new JdbcDynamicTableSink( + jdbcOptions, + getJdbcExecutionOptions(helper.getOptions()), + getJdbcDmlOptions(jdbcOptions, context.getCatalogTable().getSchema()), + rowDataType, + fieldDataTypes); + } + + @Override + public DynamicTableSource createDynamicTableSource(Context context) { + final FactoryUtil.TableFactoryHelper helper = FactoryUtil.createTableFactoryHelper(this, context); + helper.validate(); + TableSchema formatSchema = TableSchemaUtils.getPhysicalSchema(context.getCatalogTable().getSchema()); + + int[] selectFields = new int[formatSchema.getFieldNames().length]; + for (int i = 0; i < selectFields.length; i++) { + selectFields[i] = i; + } + return new JdbcDynamicTableSource( + getJdbcOptions(helper.getOptions()), + getJdbcReadOptions(helper.getOptions()), + getJdbcLookupOptions(helper.getOptions()), + formatSchema, + selectFields); + } + + private JdbcOptions getJdbcOptions(ReadableConfig readableConfig) { + final String url = readableConfig.get(URL); + final JdbcOptions.Builder builder = JdbcOptions.builder() + .setDBUrl(url) + .setTableName(readableConfig.get(TABLE)) + .setDialect(JdbcDialects.get(url).get()); + + readableConfig.getOptional(DRIVER).ifPresent(builder::setDriverName); + readableConfig.getOptional(USERNAME).ifPresent(builder::setUsername); + readableConfig.getOptional(PASSWORD).ifPresent(builder::setPassword); + return builder.build(); + } + + private JdbcReadOptions getJdbcReadOptions(ReadableConfig readableConfig) { + final Optional<String> partitionColumnName = readableConfig.getOptional(READ_PARTITION_COLUMN); + final Optional<Long> partitionLower = readableConfig.getOptional(READ_PARTITION_LOWER_BOUND); + final Optional<Long> partitionUpper = readableConfig.getOptional(READ_PARTITION_UPPER_BOUND); + final Optional<Integer> numPartitions = readableConfig.getOptional(READ_PARTITION_NUM); + + final JdbcReadOptions.Builder builder = JdbcReadOptions.builder(); + if (partitionColumnName.isPresent()) { Review comment: Add a validation that other 3 options shouldn't be empty. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSource.java ########## @@ -0,0 +1,162 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.api.java.typeutils.RowTypeInfo; +import org.apache.flink.connector.jdbc.dialect.JdbcDialect; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.connector.jdbc.split.JdbcNumericBetweenParametersProvider; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.ChangelogMode; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.connector.source.InputFormatProvider; +import org.apache.flink.table.connector.source.LookupTableSource; +import org.apache.flink.table.connector.source.ScanTableSource; +import org.apache.flink.table.connector.source.TableFunctionProvider; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.types.logical.LogicalType; +import org.apache.flink.table.types.logical.RowType; + +import java.util.Arrays; +import java.util.Objects; + +import static org.apache.flink.table.types.utils.TypeConversions.fromDataTypeToLegacyInfo; + +/** + * A {@link DynamicTableSource} for JDBC. + */ +@Internal +public class JdbcDynamicTableSource implements ScanTableSource, LookupTableSource { + + private static final String name = "JdbcTableSource"; + private final JdbcOptions options; + private final JdbcReadOptions readOptions; + private final JdbcLookupOptions lookupOptions; + private final TableSchema schema; + private final int[] selectFields; + + public JdbcDynamicTableSource( + JdbcOptions options, + JdbcReadOptions readOptions, + JdbcLookupOptions lookupOptions, + TableSchema schema, + int[] selectFields) { + this.options = options; + this.readOptions = readOptions; + this.lookupOptions = lookupOptions; + this.schema = schema; + this.selectFields = selectFields; + } + + @Override + public LookupRuntimeProvider getLookupRuntimeProvider(LookupTableSource.Context context) { + // JDBC only support non-nested look up keys + String[] keyNames = new String[context.getKeys().length]; + for (int i = 0; i < keyNames.length; i++) { + int index = context.getKeys()[i][0]; + keyNames[i] = schema.getFieldNames()[index]; + } + return TableFunctionProvider.of(JdbcDynamicLookupFunction.builder() + .setFieldNames(schema.getFieldNames()) + .setFieldTypes(schema.getFieldDataTypes()) + .setKeyNames(keyNames) + .setOptions(options) + .setLookupOptions(lookupOptions) + .build()); + } + + @Override + public ScanRuntimeProvider getScanRuntimeProvider(ScanTableSource.Context runtimeProviderContext) { + final DataType rowDataType = schema.toPhysicalRowDataType(); + final RowTypeInfo rowTypeInfo = (RowTypeInfo) fromDataTypeToLegacyInfo(rowDataType); + final JdbcDynamicInputFormat.Builder builder = JdbcDynamicInputFormat.builder() + .setDrivername(options.getDriverName()) + .setDBUrl(options.getDbURL()) + .setUsername(options.getUsername().orElse(null)) + .setPassword(options.getPassword().orElse(null)) + .setRowTypeInfo(new RowTypeInfo(rowTypeInfo.getFieldTypes(), rowTypeInfo.getFieldNames())); + + if (readOptions.getFetchSize() != 0) { + builder.setFetchSize(readOptions.getFetchSize()); + } + final JdbcDialect dialect = options.getDialect(); + String query = dialect.getSelectFromStatement( + options.getTableName(), rowTypeInfo.getFieldNames(), new String[0]); + if (readOptions.getPartitionColumnName().isPresent()) { + long lowerBound = readOptions.getPartitionLowerBound().get(); + long upperBound = readOptions.getPartitionUpperBound().get(); + int numPartitions = readOptions.getNumPartitions().get(); + builder.setParametersProvider( + new JdbcNumericBetweenParametersProvider(lowerBound, upperBound).ofBatchNum(numPartitions)); + query += " WHERE " + + dialect.quoteIdentifier(readOptions.getPartitionColumnName().get()) + + " BETWEEN ? AND ?"; + } + builder.setQuery(query); + final RowType rowType = RowType.of( + Arrays.stream(schema.getFieldDataTypes()) + .map(DataType::getLogicalType) + .toArray(LogicalType[]::new), + schema.getFieldNames()); Review comment: final RowType rowType = (RowType) schema.toRowDataType().getLogicalType(); ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSourceSinkFactory.java ########## @@ -0,0 +1,283 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.configuration.ConfigOption; +import org.apache.flink.configuration.ConfigOptions; +import org.apache.flink.configuration.ReadableConfig; +import org.apache.flink.connector.jdbc.JdbcExecutionOptions; +import org.apache.flink.connector.jdbc.dialect.JdbcDialects; +import org.apache.flink.connector.jdbc.internal.options.JdbcDmlOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.sink.DynamicTableSink; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.factories.DynamicTableSinkFactory; +import org.apache.flink.table.factories.DynamicTableSourceFactory; +import org.apache.flink.table.factories.FactoryUtil; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.utils.TableSchemaUtils; + +import java.time.Duration; +import java.util.HashSet; +import java.util.Optional; +import java.util.Set; + +/** + * Factory for creating configured instances of {@link JdbcDynamicTableSource} + * and {@link JdbcDynamicTableSink}. + */ +@Internal +public class JdbcDynamicTableSourceSinkFactory implements DynamicTableSourceFactory, DynamicTableSinkFactory { + + public static final ConfigOption<String> IDENTIFIER = ConfigOptions + .key("connector") + .stringType() + .defaultValue("jdbc") + .withDescription("-- required: specify this table type is jdbc."); + public static final ConfigOption<String> URL = ConfigOptions + .key("url") + .stringType() + .noDefaultValue() + .withDescription("-- required: the jdbc database url."); + public static final ConfigOption<String> TABLE = ConfigOptions + .key("table") + .stringType() + .noDefaultValue() + .withDescription("-- required: the jdbc table name."); + public static final ConfigOption<String> USERNAME = ConfigOptions + .key("username") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the jdbc user name."); + public static final ConfigOption<String> PASSWORD = ConfigOptions + .key("password") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the jdbc password."); + private static final ConfigOption<String> DRIVER = ConfigOptions + .key("driver") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the class name of the JDBC driver to use to connect to this URL. " + + "If not set, it will automatically be derived from the URL."); + + // read config options + private static final ConfigOption<String> READ_PARTITION_COLUMN = ConfigOptions + .key("read.partition.column") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the column name used for partitioning the input."); + private static final ConfigOption<Integer> READ_PARTITION_NUM = ConfigOptions + .key("read.partition.num") + .intType() + .noDefaultValue() + .withDescription("-- optional: the number of partitions."); + private static final ConfigOption<Long> READ_PARTITION_LOWER_BOUND = ConfigOptions + .key("read.partition.lower-bound") + .longType() + .noDefaultValue() + .withDescription("-- optional: the smallest value of the first partition."); + private static final ConfigOption<Long> READ_PARTITION_UPPER_BOUND = ConfigOptions + .key("read.partition.upper-bound") + .longType() + .noDefaultValue() + .withDescription("-- optional: the largest value of the last partition."); + private static final ConfigOption<Integer> READ_FETCH_SIZE = ConfigOptions + .key("read.fetch-size") + .intType() + .defaultValue(0) + .withDescription("-- optional, Gives the reader a hint as to the number of rows that should be fetched, from" + + " the database when reading per round trip. If the value specified is zero, then the hint is ignored. The" + + " default value is zero."); + + // look up config options + private static final ConfigOption<Long> LOOKUP_CACHE_MAX_ROWS = ConfigOptions + .key("lookup.cache.max-rows") + .longType() + .defaultValue(-1L) + .withDescription("-- optional, max number of rows of lookup cache, over this value, the oldest rows will " + + "be eliminated. \"cache.max-rows\" and \"cache.ttl\" options must all be specified if any of them is " + + "specified. Cache is not enabled as default."); + private static final ConfigOption<Duration> LOOKUP_CACHE_TTL = ConfigOptions + .key("lookup.cache.ttl") + .durationType() + .defaultValue(Duration.ofSeconds(10)) + .withDescription("-- optional, the cache time to live."); + private static final ConfigOption<Integer> LOOKUP_MAX_RETRIES = ConfigOptions + .key("lookup.max-retries") + .intType() + .defaultValue(3) + .withDescription("-- optional, max retry times if lookup database failed."); + + // write config options + private static final ConfigOption<Integer> WRITE_FLUSH_MAX_ROWS = ConfigOptions + .key("write.flush.max-rows") + .intType() + .defaultValue(5000) + .withDescription("-- optional, flush max size (includes all append, upsert and delete records), over this number" + + " of records, will flush data. The default value is 5000."); + private static final ConfigOption<Long> WRITE_FLUSH_INTERVAL = ConfigOptions + .key("write.flush.interval") + .longType() + .defaultValue(0L) + .withDescription("-- optional, flush interval mills, over this time, asynchronous threads will flush data. The " + + "default value is 0, which means no asynchronous flush thread will be scheduled."); + private static final ConfigOption<Integer> WRITE_MAX_RETRIES = ConfigOptions + .key("write.max-retries") + .intType() + .defaultValue(3) + .withDescription("-- optional, max retry times if writing records to database failed."); + + @Override + public DynamicTableSink createDynamicTableSink(Context context) { + final FactoryUtil.TableFactoryHelper helper = FactoryUtil.createTableFactoryHelper(this, context); + helper.validate(); + final JdbcOptions jdbcOptions = getJdbcOptions(helper.getOptions()); + final DataType rowDataType = context.getCatalogTable().getSchema().toPhysicalRowDataType(); + final TableSchema formatSchema = TableSchemaUtils.getPhysicalSchema(context.getCatalogTable().getSchema()); + final DataType[] fieldDataTypes = formatSchema.getFieldDataTypes(); Review comment: We don't need to declare every local field as `final`. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSourceSinkFactory.java ########## @@ -0,0 +1,283 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.configuration.ConfigOption; +import org.apache.flink.configuration.ConfigOptions; +import org.apache.flink.configuration.ReadableConfig; +import org.apache.flink.connector.jdbc.JdbcExecutionOptions; +import org.apache.flink.connector.jdbc.dialect.JdbcDialects; +import org.apache.flink.connector.jdbc.internal.options.JdbcDmlOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.sink.DynamicTableSink; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.factories.DynamicTableSinkFactory; +import org.apache.flink.table.factories.DynamicTableSourceFactory; +import org.apache.flink.table.factories.FactoryUtil; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.utils.TableSchemaUtils; + +import java.time.Duration; +import java.util.HashSet; +import java.util.Optional; +import java.util.Set; + +/** + * Factory for creating configured instances of {@link JdbcDynamicTableSource} + * and {@link JdbcDynamicTableSink}. + */ +@Internal +public class JdbcDynamicTableSourceSinkFactory implements DynamicTableSourceFactory, DynamicTableSinkFactory { + + public static final ConfigOption<String> IDENTIFIER = ConfigOptions + .key("connector") + .stringType() + .defaultValue("jdbc") + .withDescription("-- required: specify this table type is jdbc."); + public static final ConfigOption<String> URL = ConfigOptions + .key("url") + .stringType() + .noDefaultValue() + .withDescription("-- required: the jdbc database url."); + public static final ConfigOption<String> TABLE = ConfigOptions + .key("table") + .stringType() + .noDefaultValue() + .withDescription("-- required: the jdbc table name."); + public static final ConfigOption<String> USERNAME = ConfigOptions + .key("username") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the jdbc user name."); + public static final ConfigOption<String> PASSWORD = ConfigOptions + .key("password") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the jdbc password."); + private static final ConfigOption<String> DRIVER = ConfigOptions + .key("driver") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the class name of the JDBC driver to use to connect to this URL. " + + "If not set, it will automatically be derived from the URL."); + + // read config options + private static final ConfigOption<String> READ_PARTITION_COLUMN = ConfigOptions + .key("read.partition.column") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the column name used for partitioning the input."); + private static final ConfigOption<Integer> READ_PARTITION_NUM = ConfigOptions + .key("read.partition.num") + .intType() + .noDefaultValue() + .withDescription("-- optional: the number of partitions."); + private static final ConfigOption<Long> READ_PARTITION_LOWER_BOUND = ConfigOptions + .key("read.partition.lower-bound") + .longType() + .noDefaultValue() + .withDescription("-- optional: the smallest value of the first partition."); + private static final ConfigOption<Long> READ_PARTITION_UPPER_BOUND = ConfigOptions + .key("read.partition.upper-bound") + .longType() + .noDefaultValue() + .withDescription("-- optional: the largest value of the last partition."); + private static final ConfigOption<Integer> READ_FETCH_SIZE = ConfigOptions + .key("read.fetch-size") + .intType() + .defaultValue(0) + .withDescription("-- optional, Gives the reader a hint as to the number of rows that should be fetched, from" + + " the database when reading per round trip. If the value specified is zero, then the hint is ignored. The" + + " default value is zero."); + + // look up config options + private static final ConfigOption<Long> LOOKUP_CACHE_MAX_ROWS = ConfigOptions + .key("lookup.cache.max-rows") + .longType() + .defaultValue(-1L) + .withDescription("-- optional, max number of rows of lookup cache, over this value, the oldest rows will " + + "be eliminated. \"cache.max-rows\" and \"cache.ttl\" options must all be specified if any of them is " + + "specified. Cache is not enabled as default."); + private static final ConfigOption<Duration> LOOKUP_CACHE_TTL = ConfigOptions + .key("lookup.cache.ttl") + .durationType() + .defaultValue(Duration.ofSeconds(10)) + .withDescription("-- optional, the cache time to live."); + private static final ConfigOption<Integer> LOOKUP_MAX_RETRIES = ConfigOptions + .key("lookup.max-retries") + .intType() + .defaultValue(3) + .withDescription("-- optional, max retry times if lookup database failed."); + + // write config options + private static final ConfigOption<Integer> WRITE_FLUSH_MAX_ROWS = ConfigOptions + .key("write.flush.max-rows") + .intType() + .defaultValue(5000) + .withDescription("-- optional, flush max size (includes all append, upsert and delete records), over this number" + + " of records, will flush data. The default value is 5000."); + private static final ConfigOption<Long> WRITE_FLUSH_INTERVAL = ConfigOptions + .key("write.flush.interval") + .longType() + .defaultValue(0L) + .withDescription("-- optional, flush interval mills, over this time, asynchronous threads will flush data. The " + + "default value is 0, which means no asynchronous flush thread will be scheduled."); + private static final ConfigOption<Integer> WRITE_MAX_RETRIES = ConfigOptions + .key("write.max-retries") + .intType() + .defaultValue(3) + .withDescription("-- optional, max retry times if writing records to database failed."); + + @Override + public DynamicTableSink createDynamicTableSink(Context context) { + final FactoryUtil.TableFactoryHelper helper = FactoryUtil.createTableFactoryHelper(this, context); + helper.validate(); + final JdbcOptions jdbcOptions = getJdbcOptions(helper.getOptions()); + final DataType rowDataType = context.getCatalogTable().getSchema().toPhysicalRowDataType(); + final TableSchema formatSchema = TableSchemaUtils.getPhysicalSchema(context.getCatalogTable().getSchema()); + final DataType[] fieldDataTypes = formatSchema.getFieldDataTypes(); + + return new JdbcDynamicTableSink( + jdbcOptions, + getJdbcExecutionOptions(helper.getOptions()), + getJdbcDmlOptions(jdbcOptions, context.getCatalogTable().getSchema()), + rowDataType, + fieldDataTypes); + } + + @Override + public DynamicTableSource createDynamicTableSource(Context context) { + final FactoryUtil.TableFactoryHelper helper = FactoryUtil.createTableFactoryHelper(this, context); + helper.validate(); + TableSchema formatSchema = TableSchemaUtils.getPhysicalSchema(context.getCatalogTable().getSchema()); + + int[] selectFields = new int[formatSchema.getFieldNames().length]; + for (int i = 0; i < selectFields.length; i++) { + selectFields[i] = i; + } + return new JdbcDynamicTableSource( + getJdbcOptions(helper.getOptions()), + getJdbcReadOptions(helper.getOptions()), + getJdbcLookupOptions(helper.getOptions()), + formatSchema, + selectFields); + } + + private JdbcOptions getJdbcOptions(ReadableConfig readableConfig) { + final String url = readableConfig.get(URL); + final JdbcOptions.Builder builder = JdbcOptions.builder() + .setDBUrl(url) + .setTableName(readableConfig.get(TABLE)) + .setDialect(JdbcDialects.get(url).get()); + + readableConfig.getOptional(DRIVER).ifPresent(builder::setDriverName); + readableConfig.getOptional(USERNAME).ifPresent(builder::setUsername); + readableConfig.getOptional(PASSWORD).ifPresent(builder::setPassword); + return builder.build(); + } + + private JdbcReadOptions getJdbcReadOptions(ReadableConfig readableConfig) { + final Optional<String> partitionColumnName = readableConfig.getOptional(READ_PARTITION_COLUMN); + final Optional<Long> partitionLower = readableConfig.getOptional(READ_PARTITION_LOWER_BOUND); + final Optional<Long> partitionUpper = readableConfig.getOptional(READ_PARTITION_UPPER_BOUND); + final Optional<Integer> numPartitions = readableConfig.getOptional(READ_PARTITION_NUM); Review comment: You can put these 3 in the `partitionColumnName.isPresent()` branch block, and use `config.get(..)` to avoid the warning. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSourceSinkFactory.java ########## @@ -0,0 +1,283 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.configuration.ConfigOption; +import org.apache.flink.configuration.ConfigOptions; +import org.apache.flink.configuration.ReadableConfig; +import org.apache.flink.connector.jdbc.JdbcExecutionOptions; +import org.apache.flink.connector.jdbc.dialect.JdbcDialects; +import org.apache.flink.connector.jdbc.internal.options.JdbcDmlOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.sink.DynamicTableSink; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.factories.DynamicTableSinkFactory; +import org.apache.flink.table.factories.DynamicTableSourceFactory; +import org.apache.flink.table.factories.FactoryUtil; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.utils.TableSchemaUtils; + +import java.time.Duration; +import java.util.HashSet; +import java.util.Optional; +import java.util.Set; + +/** + * Factory for creating configured instances of {@link JdbcDynamicTableSource} + * and {@link JdbcDynamicTableSink}. + */ +@Internal +public class JdbcDynamicTableSourceSinkFactory implements DynamicTableSourceFactory, DynamicTableSinkFactory { + + public static final ConfigOption<String> IDENTIFIER = ConfigOptions Review comment: `IDENTIFIER` can be just a simple string. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSourceSinkFactory.java ########## @@ -0,0 +1,283 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.configuration.ConfigOption; +import org.apache.flink.configuration.ConfigOptions; +import org.apache.flink.configuration.ReadableConfig; +import org.apache.flink.connector.jdbc.JdbcExecutionOptions; +import org.apache.flink.connector.jdbc.dialect.JdbcDialects; +import org.apache.flink.connector.jdbc.internal.options.JdbcDmlOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.sink.DynamicTableSink; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.factories.DynamicTableSinkFactory; +import org.apache.flink.table.factories.DynamicTableSourceFactory; +import org.apache.flink.table.factories.FactoryUtil; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.utils.TableSchemaUtils; + +import java.time.Duration; +import java.util.HashSet; +import java.util.Optional; +import java.util.Set; + +/** + * Factory for creating configured instances of {@link JdbcDynamicTableSource} + * and {@link JdbcDynamicTableSink}. + */ +@Internal +public class JdbcDynamicTableSourceSinkFactory implements DynamicTableSourceFactory, DynamicTableSinkFactory { Review comment: Please add a dedicate validation logic to validate options, e.g. dialect, primary key. username and password should set together, partition options should be set together, lowerBound should < upperBound. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSourceSinkFactory.java ########## @@ -0,0 +1,283 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.configuration.ConfigOption; +import org.apache.flink.configuration.ConfigOptions; +import org.apache.flink.configuration.ReadableConfig; +import org.apache.flink.connector.jdbc.JdbcExecutionOptions; +import org.apache.flink.connector.jdbc.dialect.JdbcDialects; +import org.apache.flink.connector.jdbc.internal.options.JdbcDmlOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.sink.DynamicTableSink; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.factories.DynamicTableSinkFactory; +import org.apache.flink.table.factories.DynamicTableSourceFactory; +import org.apache.flink.table.factories.FactoryUtil; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.utils.TableSchemaUtils; + +import java.time.Duration; +import java.util.HashSet; +import java.util.Optional; +import java.util.Set; + +/** + * Factory for creating configured instances of {@link JdbcDynamicTableSource} + * and {@link JdbcDynamicTableSink}. + */ +@Internal +public class JdbcDynamicTableSourceSinkFactory implements DynamicTableSourceFactory, DynamicTableSinkFactory { + + public static final ConfigOption<String> IDENTIFIER = ConfigOptions + .key("connector") + .stringType() + .defaultValue("jdbc") + .withDescription("-- required: specify this table type is jdbc."); + public static final ConfigOption<String> URL = ConfigOptions + .key("url") + .stringType() + .noDefaultValue() + .withDescription("-- required: the jdbc database url."); + public static final ConfigOption<String> TABLE = ConfigOptions + .key("table") + .stringType() + .noDefaultValue() + .withDescription("-- required: the jdbc table name."); + public static final ConfigOption<String> USERNAME = ConfigOptions + .key("username") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the jdbc user name."); + public static final ConfigOption<String> PASSWORD = ConfigOptions + .key("password") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the jdbc password."); + private static final ConfigOption<String> DRIVER = ConfigOptions + .key("driver") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the class name of the JDBC driver to use to connect to this URL. " + + "If not set, it will automatically be derived from the URL."); + + // read config options + private static final ConfigOption<String> READ_PARTITION_COLUMN = ConfigOptions + .key("read.partition.column") + .stringType() + .noDefaultValue() + .withDescription("-- optional: the column name used for partitioning the input."); + private static final ConfigOption<Integer> READ_PARTITION_NUM = ConfigOptions + .key("read.partition.num") + .intType() + .noDefaultValue() + .withDescription("-- optional: the number of partitions."); + private static final ConfigOption<Long> READ_PARTITION_LOWER_BOUND = ConfigOptions + .key("read.partition.lower-bound") + .longType() + .noDefaultValue() + .withDescription("-- optional: the smallest value of the first partition."); + private static final ConfigOption<Long> READ_PARTITION_UPPER_BOUND = ConfigOptions + .key("read.partition.upper-bound") + .longType() + .noDefaultValue() + .withDescription("-- optional: the largest value of the last partition."); + private static final ConfigOption<Integer> READ_FETCH_SIZE = ConfigOptions + .key("read.fetch-size") + .intType() + .defaultValue(0) + .withDescription("-- optional, Gives the reader a hint as to the number of rows that should be fetched, from" + + " the database when reading per round trip. If the value specified is zero, then the hint is ignored. The" + + " default value is zero."); + + // look up config options + private static final ConfigOption<Long> LOOKUP_CACHE_MAX_ROWS = ConfigOptions + .key("lookup.cache.max-rows") + .longType() + .defaultValue(-1L) + .withDescription("-- optional, max number of rows of lookup cache, over this value, the oldest rows will " + + "be eliminated. \"cache.max-rows\" and \"cache.ttl\" options must all be specified if any of them is " + + "specified. Cache is not enabled as default."); + private static final ConfigOption<Duration> LOOKUP_CACHE_TTL = ConfigOptions + .key("lookup.cache.ttl") + .durationType() + .defaultValue(Duration.ofSeconds(10)) + .withDescription("-- optional, the cache time to live."); + private static final ConfigOption<Integer> LOOKUP_MAX_RETRIES = ConfigOptions + .key("lookup.max-retries") + .intType() + .defaultValue(3) + .withDescription("-- optional, max retry times if lookup database failed."); + + // write config options + private static final ConfigOption<Integer> WRITE_FLUSH_MAX_ROWS = ConfigOptions + .key("write.flush.max-rows") + .intType() + .defaultValue(5000) + .withDescription("-- optional, flush max size (includes all append, upsert and delete records), over this number" + + " of records, will flush data. The default value is 5000."); + private static final ConfigOption<Long> WRITE_FLUSH_INTERVAL = ConfigOptions + .key("write.flush.interval") + .longType() + .defaultValue(0L) + .withDescription("-- optional, flush interval mills, over this time, asynchronous threads will flush data. The " + + "default value is 0, which means no asynchronous flush thread will be scheduled."); + private static final ConfigOption<Integer> WRITE_MAX_RETRIES = ConfigOptions + .key("write.max-retries") + .intType() + .defaultValue(3) + .withDescription("-- optional, max retry times if writing records to database failed."); + + @Override + public DynamicTableSink createDynamicTableSink(Context context) { Review comment: Add a validation that JDBC sink table should declare a primary key. ########## File path: flink-connectors/flink-connector-jdbc/src/main/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSource.java ########## @@ -0,0 +1,162 @@ +/* + * 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.flink.connector.jdbc.table; + +import org.apache.flink.annotation.Internal; +import org.apache.flink.api.java.typeutils.RowTypeInfo; +import org.apache.flink.connector.jdbc.dialect.JdbcDialect; +import org.apache.flink.connector.jdbc.internal.options.JdbcLookupOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcOptions; +import org.apache.flink.connector.jdbc.internal.options.JdbcReadOptions; +import org.apache.flink.connector.jdbc.split.JdbcNumericBetweenParametersProvider; +import org.apache.flink.table.api.TableSchema; +import org.apache.flink.table.connector.ChangelogMode; +import org.apache.flink.table.connector.source.DynamicTableSource; +import org.apache.flink.table.connector.source.InputFormatProvider; +import org.apache.flink.table.connector.source.LookupTableSource; +import org.apache.flink.table.connector.source.ScanTableSource; +import org.apache.flink.table.connector.source.TableFunctionProvider; +import org.apache.flink.table.types.DataType; +import org.apache.flink.table.types.logical.LogicalType; +import org.apache.flink.table.types.logical.RowType; + +import java.util.Arrays; +import java.util.Objects; + +import static org.apache.flink.table.types.utils.TypeConversions.fromDataTypeToLegacyInfo; + +/** + * A {@link DynamicTableSource} for JDBC. + */ +@Internal +public class JdbcDynamicTableSource implements ScanTableSource, LookupTableSource { + + private static final String name = "JdbcTableSource"; + private final JdbcOptions options; + private final JdbcReadOptions readOptions; + private final JdbcLookupOptions lookupOptions; + private final TableSchema schema; + private final int[] selectFields; + + public JdbcDynamicTableSource( + JdbcOptions options, + JdbcReadOptions readOptions, + JdbcLookupOptions lookupOptions, + TableSchema schema, + int[] selectFields) { + this.options = options; + this.readOptions = readOptions; + this.lookupOptions = lookupOptions; + this.schema = schema; + this.selectFields = selectFields; + } + + @Override + public LookupRuntimeProvider getLookupRuntimeProvider(LookupTableSource.Context context) { + // JDBC only support non-nested look up keys + String[] keyNames = new String[context.getKeys().length]; + for (int i = 0; i < keyNames.length; i++) { + int index = context.getKeys()[i][0]; + keyNames[i] = schema.getFieldNames()[index]; + } + return TableFunctionProvider.of(JdbcDynamicLookupFunction.builder() + .setFieldNames(schema.getFieldNames()) + .setFieldTypes(schema.getFieldDataTypes()) + .setKeyNames(keyNames) + .setOptions(options) + .setLookupOptions(lookupOptions) + .build()); + } + + @Override + public ScanRuntimeProvider getScanRuntimeProvider(ScanTableSource.Context runtimeProviderContext) { + final DataType rowDataType = schema.toPhysicalRowDataType(); + final RowTypeInfo rowTypeInfo = (RowTypeInfo) fromDataTypeToLegacyInfo(rowDataType); + final JdbcDynamicInputFormat.Builder builder = JdbcDynamicInputFormat.builder() + .setDrivername(options.getDriverName()) + .setDBUrl(options.getDbURL()) + .setUsername(options.getUsername().orElse(null)) + .setPassword(options.getPassword().orElse(null)) + .setRowTypeInfo(new RowTypeInfo(rowTypeInfo.getFieldTypes(), rowTypeInfo.getFieldNames())); Review comment: This is never used. ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: [email protected]
