[FLINK-38729] Add Flink2 support for Source/Pipeline connector.#4322
Open
lvyanquan wants to merge 10 commits intoapache:masterfrom
Open
[FLINK-38729] Add Flink2 support for Source/Pipeline connector.#4322lvyanquan wants to merge 10 commits intoapache:masterfrom
lvyanquan wants to merge 10 commits intoapache:masterfrom
Conversation
637b01b to
f1f09e6
Compare
yuxiqian
reviewed
Mar 18, 2026
Member
yuxiqian
left a comment
There was a problem hiding this comment.
Thanks Kunni for the marvelous work! Just left some comments.
...omposer/src/main/java/org/apache/flink/cdc/composer/flink/translator/DataSinkTranslator.java
Outdated
Show resolved
Hide resolved
...omposer/src/main/java/org/apache/flink/cdc/composer/flink/translator/DataSinkTranslator.java
Outdated
Show resolved
Hide resolved
...ava/org/apache/flink/cdc/connectors/elasticsearch/sink/Elasticsearch6DataSinkITCaseTest.java
Outdated
Show resolved
Hide resolved
...eline-connectors/flink-cdc-pipeline-connector-hudi/src/test/resources/log4j2-test.properties
Outdated
Show resolved
Hide resolved
flink-cdc-flink1-compat/src/main/java/org/apache/flink/streaming/util/RestartStrategyUtils.java
Show resolved
Hide resolved
...src/test/java/org/apache/flink/cdc/composer/testsource/source/DistributedSourceFunction.java
Show resolved
Hide resolved
...-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/table/MongoDBConnectorITCase.java
Outdated
Show resolved
Hide resolved
...ongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/utils/MongoDBTestUtils.java
Outdated
Show resolved
Hide resolved
...c-source-connectors/flink-connector-postgres-cdc/src/test/resources/ddl/replica_identity.sql
Outdated
Show resolved
Hide resolved
Contributor
Author
|
|
yuxiqian
reviewed
Mar 19, 2026
...ris/src/test/java/org/apache/flink/cdc/connectors/doris/sink/DorisMetadataApplierITCase.java
Outdated
Show resolved
Hide resolved
….x builds." This reverts commit 49245a7.
Contributor
There was a problem hiding this comment.
Pull request overview
This PR extends Flink CDC source and pipeline connectors to build and run against Flink 2.x via a -Pflink2 Maven profile, adding/adjusting compatibility shims and updating CI to execute Flink 2.x test suites.
Changes:
- Introduces Flink 1.x / 2.x compatibility adapters (table factory utils, sink/source API shims, runtime/test adapters).
- Updates multiple connectors and tests to compile/run on Flink 2.x (restart strategies, savepoint APIs, collect sink handling, serializer config changes).
- Updates build/CI wiring and dependency coordinates to select Flink-2-compatible artifacts (Kafka/Iceberg/Paimon, etc.).
Reviewed changes
Copilot reviewed 143 out of 143 changed files in this pull request and generated 1 comment.
Show a summary per file
| File | Description |
|---|---|
| pom.xml | Adds version properties and Flink2 profile overrides; propagates flink.profile=flink2 to tests and stabilizes extra-lib jar naming. |
| flink-cdc-flink2-compat/src/test/java/org/apache/flink/cdc/runtime/compat/OperatorStateStoreAdapter.java | Test adapter to cover new Flink 2.x OperatorStateStore v2 descriptor methods. |
| flink-cdc-flink2-compat/src/test/java/org/apache/flink/cdc/runtime/compat/MockStreamingRuntimeContextAdapter.java | Test factory to adapt MockStreamingRuntimeContext ctor signature change in Flink 2.x. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/table/factories/FactoryUtilAdapter.java | Adapter bridging FactoryUtil table source creation API changes for Flink 2.x. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/table/connector/source/SourceFunctionProvider.java | Legacy SourceFunction provider shim for Flink 2.x table connector API. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/table/catalog/CatalogTableAdapter.java | Adapter for CatalogTable factory signature differences (builder vs of). |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/table/api/TableSchema.java | Legacy TableSchema shim for Flink 2.x compatibility. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/streaming/api/operators/collect/CollectResultIteratorAdapter.java | Adapter to align Collect iterator construction/UID expectations in Flink 2.x. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/streaming/api/functions/source/SourceFunction.java | Compatibility SourceFunction wrapper bridging legacy/new SourceContext calls. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/streaming/api/functions/source/RichSourceFunction.java | RichSourceFunction shim for Flink 2.x OpenContext signature change. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/streaming/api/functions/source/RichParallelSourceFunction.java | Delegates legacy run to the compat default implementation for Flink 2.x. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/streaming/api/functions/sink/RichSinkFunction.java | RichSinkFunction shim for Flink 2.x OpenContext signature change. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/runtime/jobgraph/SavepointConfigOptions.java | Removes SavepointConfigOptions shim in favor of newer config options. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/api/connector/sink2/TwoPhaseCommittingSink.java | Adds default bridging between legacy and Flink 2.x committer creation APIs. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/api/connector/sink2/StatefulSink.java | StatefulSink shim aligning legacy/new writer restore signatures. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/api/connector/sink2/Sink.java | Adds WriterInitContext-to-InitContext adapter path and TaskInfo helpers. |
| flink-cdc-flink2-compat/src/main/java/org/apache/flink/api/connector/sink2/InitContextAdapter.java | Adapts WriterInitContext into legacy InitContext for older sink implementations. |
| flink-cdc-flink2-compat/pom.xml | Adds provided/test dependencies and publishes test-jar for reuse. |
| flink-cdc-flink1-compat/src/test/java/org/apache/flink/cdc/runtime/compat/OperatorStateStoreAdapter.java | Flink 1.x no-op OperatorStateStore adapter for shared test code. |
| flink-cdc-flink1-compat/src/test/java/org/apache/flink/cdc/runtime/compat/MockStreamingRuntimeContextAdapter.java | Flink 1.x MockStreamingRuntimeContext factory preserving old ctor order. |
| flink-cdc-flink1-compat/src/main/java/org/apache/flink/table/factories/FactoryUtilAdapter.java | Flink 1.x FactoryUtil wrapper for unified test code paths. |
| flink-cdc-flink1-compat/src/main/java/org/apache/flink/table/catalog/CatalogTableAdapter.java | Flink 1.x CatalogTable factory wrapper for unified test code paths. |
| flink-cdc-flink1-compat/src/main/java/org/apache/flink/streaming/util/RestartStrategyUtils.java | Flink 1.x restart strategy helper to replace direct RestartStrategies usage. |
| flink-cdc-flink1-compat/src/main/java/org/apache/flink/streaming/api/operators/collect/CollectResultIteratorAdapter.java | Flink 1.x Collect iterator adapter using operatorIdFuture. |
| flink-cdc-flink1-compat/pom.xml | Adds provided/test deps and publishes test-jar for reuse. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/specs/FlinkPipelineSpecsITCase.java | Removes Flink-1-only system property gating to allow Flink 2 runs. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/UdfE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/TransformE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/SchemaEvolvingTransformE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/SchemaEvolveE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/RouteE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/PostgresE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/OracleE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/MysqlToKafkaE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/MysqlE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/MySqlToPaimonE2eITCase.java | Parameterizes pipeline parallelism in YAML. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/MySqlToIcebergE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/MaxComputeE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/src/test/java/org/apache/flink/cdc/pipeline/tests/FlussE2eITCase.java | Removes Flink-1-only system property gating. |
| flink-cdc-e2e-tests/flink-cdc-pipeline-e2e-tests/pom.xml | Updates Paimon version, makes Iceberg artifactId/profile-driven, adds Flink2 connector jars. |
| flink-cdc-connect/pom.xml | Adds Flink1/Flink2 profiles, wiring compat test-jars and extra-libs for Flink2 test runs. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-vitess-cdc/src/test/java/org/apache/flink/cdc/connectors/vitess/table/VitessTableFactoryTest.java | Switches to CatalogTable/FactoryUtil adapters for cross-Flink compatibility. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-vitess-cdc/src/test/java/org/apache/flink/cdc/connectors/vitess/VitessSourceTest.java | Uses runtime/test adapters for MockStreamingRuntimeContext and OperatorStateStore. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-tidb-cdc/src/test/java/org/apache/flink/cdc/connectors/tidb/table/TiDBTableSourceFactoryTest.java | Switches to CatalogTable/FactoryUtil adapters for cross-Flink compatibility. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-tidb-cdc/src/main/java/org/apache/flink/cdc/connectors/tidb/TiKVRichParallelSourceFunction.java | Uses legacy SourceFunction.SourceContext and TaskInfo API to align with Flink 2.x. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-sqlserver-cdc/src/test/java/org/apache/flink/cdc/connectors/sqlserver/table/SqlServerTableFactoryTest.java | Switches to CatalogTable/FactoryUtil adapters for cross-Flink compatibility. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-sqlserver-cdc/src/test/java/org/apache/flink/cdc/connectors/sqlserver/source/SqlServerSourceITCase.java | Replaces RestartStrategies usage with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-postgres-cdc/src/test/java/org/apache/flink/cdc/connectors/postgres/table/PostgreSQLTableFactoryTest.java | Switches to CatalogTable/FactoryUtil adapters for cross-Flink compatibility. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-postgres-cdc/src/test/java/org/apache/flink/cdc/connectors/postgres/table/PostgreSQLSavepointITCase.java | Migrates savepoint config/options and restart strategy utilities; updates triggerSavepoint API. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-postgres-cdc/src/test/java/org/apache/flink/cdc/connectors/postgres/source/NewlyAddedTableITCase.java | Disables for Flink2 profile; migrates savepoint + restart utilities and triggerSavepoint API. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-postgres-cdc/src/test/java/org/apache/flink/cdc/connectors/postgres/PostgresTestBase.java | Adds timeout-based wait helpers to reduce hanging tests. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-postgres-cdc/src/test/java/org/apache/flink/cdc/connectors/postgres/PostgreSQLSourceTest.java | Uses runtime/test adapters for MockStreamingRuntimeContext and OperatorStateStore. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-oracle-cdc/src/test/java/org/apache/flink/cdc/connectors/oracle/table/OracleTableSourceFactoryTest.java | Switches to CatalogTable/FactoryUtil adapters for cross-Flink compatibility. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-oracle-cdc/src/test/java/org/apache/flink/cdc/connectors/oracle/source/OracleSourceITCase.java | Replaces RestartStrategies usage with RestartStrategyUtils helper and boolean flags. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-oracle-cdc/src/test/java/org/apache/flink/cdc/connectors/oracle/source/NewlyAddedTableITCase.java | Disables for Flink2 profile; migrates savepoint config/options and restart utilities. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-oracle-cdc/src/test/java/org/apache/flink/cdc/connectors/oracle/OracleSourceTest.java | Uses runtime/test adapters for MockStreamingRuntimeContext and OperatorStateStore. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-oceanbase-cdc/src/test/java/org/apache/flink/cdc/connectors/oceanbase/OceanBaseSourceITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-oceanbase-cdc/src/test/java/org/apache/flink/cdc/connectors/oceanbase/OceanBaseFailoverITCase.java | Replaces RestartStrategies with RestartStrategyUtils and parameterizes restart config. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/test/java/org/apache/flink/cdc/connectors/polardbx/PolardbxSourceITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/test/java/org/apache/flink/cdc/connectors/mysql/table/MySqlTableSourceFactoryTest.java | Updates to Flink 2 table source creation API (createDynamicTableSource + builder). |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/test/java/org/apache/flink/cdc/connectors/mysql/table/MySqlConnectorITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/test/java/org/apache/flink/cdc/connectors/mysql/source/reader/MySqlSourceReaderTest.java | Updates reader construction for updated Flink source reader APIs. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/test/java/org/apache/flink/cdc/connectors/mysql/source/SpecificStartingOffsetITCase.java | Migrates savepoint config/options, serializer config usage, and Collect iterator adapter. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/test/java/org/apache/flink/cdc/connectors/mysql/source/MySqlMultipleTablesRenamingITCase.java | Migrates savepoint config/options, serializer config usage, and Collect iterator adapter. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/test/java/org/apache/flink/cdc/connectors/mysql/MySqlTestUtils.java | Uses runtime/test adapters for MockStreamingRuntimeContext and OperatorStateStore. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/main/java/org/apache/flink/cdc/connectors/mysql/source/split/MySqlSplitSerializer.java | Uses classloader-aware LogicalTypeParser for Flink 2.x. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/main/java/org/apache/flink/cdc/connectors/mysql/source/reader/MySqlSourceReader.java | Updates to new SingleThreadFetcherManager constructor signature. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/main/java/org/apache/flink/cdc/connectors/mysql/source/MySqlSource.java | Updates reader creation to match updated reader ctor signature. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/utils/TestTable.java | Makes RowData stringification stable across Flink versions (null BIGINT behavior). |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/utils/MongoDBTestUtils.java | Adds Duration-based timeouts and a sink-result wait helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/table/MongoDBTableFactoryTest.java | Switches to CatalogTable/FactoryUtil adapters for cross-Flink compatibility. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/table/MongoDBConnectorITCase.java | Adjusts assertions/waits for Flink 2 changelog behavior (but introduces an infinite wait risk). |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/source/NewlyAddedTableITCase.java | Disables for Flink2 profile; migrates savepoint config/options and restart utilities. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/source/MongoDBParallelSourceITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/source/MongoDBParallelSourceExampleTest.java | Updates checkpoint option constant import for Flink 2.x. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/source/MongoDBMetricsITCase.java | Updates serializer config usage and Collect iterator adapter. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/source/MongoDBFullChangelogITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mongodb-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/LegacyMongoDBSourceTest.java | Uses runtime/test adapters for MockStreamingRuntimeContext and OperatorStateStore. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-debezium/src/main/java/org/apache/flink/cdc/debezium/DebeziumSourceFunction.java | Switches to TaskInfo API for subtask index logging compatibility. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-db2-cdc/src/test/java/org/apache/flink/cdc/connectors/db2/table/Db2TableSourceFactoryTest.java | Switches to CatalogTable/FactoryUtil adapters for cross-Flink compatibility. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-db2-cdc/src/test/java/org/apache/flink/cdc/connectors/db2/table/Db2ConnectorITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-db2-cdc/src/test/java/org/apache/flink/cdc/connectors/db2/source/Db2SourceITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-connector-db2-cdc/src/test/java/org/apache/flink/cdc/connectors/db2/Db2SourceTest.java | Uses runtime/test adapters for MockStreamingRuntimeContext and OperatorStateStore. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-cdc-base/src/main/java/org/apache/flink/cdc/connectors/base/source/reader/IncrementalSourceReader.java | Updates to new SingleThreadFetcherManager constructor signature. |
| flink-cdc-connect/flink-cdc-source-connectors/flink-cdc-base/src/main/java/org/apache/flink/cdc/connectors/base/source/meta/split/SourceSplitSerializer.java | Uses classloader-aware LogicalTypeParser for Flink 2.x. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-values/pom.xml | Removes Flink2-specific profile in favor of top-level Flink2 build profile. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-starrocks/src/test/java/org/apache/flink/cdc/connectors/starrocks/sink/StarRocksPipelineITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-starrocks/src/test/java/org/apache/flink/cdc/connectors/starrocks/sink/StarRocksMetadataApplierITCase.java | Uses EventTypeInfo + env.fromData and RestartStrategyUtils. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-starrocks/src/test/java/org/apache/flink/cdc/connectors/starrocks/sink/EventRecordSerializationSchemaTest.java | Removes invalid @Override annotations for changed InitContext interface. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-postgres/src/test/java/org/apache/flink/cdc/connectors/postgres/source/PostgresFullTypesITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-paimon/src/main/java/org/apache/flink/cdc/connectors/paimon/sink/v2/bucket/FlushEventAlignmentOperator.java | Switches to AbstractStreamOperatorAdapter for Flink 2.x compatibility. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-paimon/src/main/java/org/apache/flink/cdc/connectors/paimon/sink/v2/bucket/BucketWrapperEventTypeInfo.java | Adds SerializerConfig-based createSerializer overload for Flink 2.x. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-paimon/src/main/java/org/apache/flink/cdc/connectors/paimon/sink/v2/bucket/BucketAssignOperator.java | Switches to AbstractStreamOperatorAdapter for Flink 2.x compatibility. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-paimon/src/main/java/org/apache/flink/cdc/connectors/paimon/sink/v2/PreCommitOperator.java | Switches to TaskInfo API for subtask/parallelism. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-paimon/src/main/java/org/apache/flink/cdc/connectors/paimon/sink/v2/PaimonWriter.java | Updates to Flink 2 sink writer interfaces (CommittingSinkWriter). |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-paimon/src/main/java/org/apache/flink/cdc/connectors/paimon/sink/v2/PaimonSink.java | Updates sink interfaces to TwoPhaseCommittingSink + SupportsPreCommitTopology and new serializers. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-paimon/src/main/java/org/apache/flink/cdc/connectors/paimon/sink/v2/PaimonEventSink.java | Updates pre-write topology interface and write-result serializer method. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-paimon/pom.xml | Uses profile-driven Paimon artifact id (paimon.flink.major.version). |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-oracle/src/test/java/org/apache/flink/cdc/connectors/oracle/source/OraclePipelineITCase.java | Migrates savepoint APIs, serializer config usage, restart strategy utils, and Collect iterator adapter. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-oracle/src/test/java/org/apache/flink/cdc/connectors/oracle/source/OracleMetadataAccessorITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-oracle/src/test/java/org/apache/flink/cdc/connectors/oracle/source/OracleFullTypesITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-mysql/src/test/java/org/apache/flink/cdc/connectors/mysql/source/MysqlPipelineNewlyAddedTableITCase.java | Migrates savepoint APIs, serializer config usage, restart strategy utils, and Collect iterator adapter. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-mysql/src/test/java/org/apache/flink/cdc/connectors/mysql/source/MySqlTableIdCaseInsensitveITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-mysql/src/test/java/org/apache/flink/cdc/connectors/mysql/source/MySqlPipelineITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-mysql/src/test/java/org/apache/flink/cdc/connectors/mysql/source/MySqlOnLineSchemaMigrationITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-mysql/src/test/java/org/apache/flink/cdc/connectors/mysql/source/MySqlMetadataAccessorITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-mysql/src/test/java/org/apache/flink/cdc/connectors/mysql/source/MySqlFullTypesITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-mysql/src/test/java/org/apache/flink/cdc/connectors/mysql/source/MySqlAncientDateAndTimeITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-maxcompute/src/main/java/org/apache/flink/cdc/connectors/maxcompute/sink/MaxComputeEventWriter.java | Adds WriterInitContext constructor and switches subtask lookup to TaskInfo API. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-maxcompute/src/main/java/org/apache/flink/cdc/connectors/maxcompute/sink/MaxComputeEventSink.java | Implements SupportsPreWriteTopology and adds WriterInitContext createWriter overload. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-maxcompute/src/main/java/org/apache/flink/cdc/connectors/maxcompute/coordinator/SessionManageOperator.java | Switches to AbstractStreamOperatorAdapter and TaskInfo API. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-kafka/src/test/java/org/apache/flink/cdc/connectors/kafka/sink/KafkaDataSinkITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-kafka/pom.xml | Makes Kafka connector version profile-driven. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-iceberg/src/test/java/org/apache/flink/cdc/connectors/iceberg/sink/v2/IcebergSinkITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-iceberg/src/main/java/org/apache/flink/cdc/connectors/iceberg/sink/v2/compaction/CompactionOperator.java | Switches to AbstractStreamOperatorAdapter for Flink 2.x compatibility. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-iceberg/src/main/java/org/apache/flink/cdc/connectors/iceberg/sink/v2/IcebergSink.java | Marks legacy createWriter as deprecated; aligns sink API expectations. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-iceberg/pom.xml | Uses profile-driven Iceberg Flink runtime artifact id. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/test/resources/log4j2-test.properties | Adds test log4j2 config to reduce CI log noise. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/test/java/org/apache/flink/cdc/connectors/hudi/sink/HudiSinkITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/main/java/org/apache/hudi/utils/CatalogUtils.java | Adds CatalogTable builder utility to avoid deprecated API usage. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/main/java/org/apache/flink/cdc/connectors/hudi/sink/util/RowDataUtils.java | Switches config accessors to ConfigOption-based Configuration#set/get. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/main/java/org/apache/flink/cdc/connectors/hudi/sink/function/MultiTableEventStreamWriteFunction.java | Migrates to OpenContext open method, TaskInfo API, and ConfigOption get. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/main/java/org/apache/flink/cdc/connectors/hudi/sink/coordinator/MultiTableStreamWriteOperatorCoordinator.java | Migrates to ConfigOption get for Flink options. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/main/java/org/apache/flink/cdc/connectors/hudi/sink/bucket/FlushEventAlignmentOperator.java | Switches to AbstractStreamOperatorAdapter for Flink 2.x compatibility. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/main/java/org/apache/flink/cdc/connectors/hudi/sink/bucket/BucketAssignOperator.java | Switches to AbstractStreamOperatorAdapter and ConfigOption get. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/src/main/java/org/apache/flink/cdc/connectors/hudi/sink/HudiMetadataApplier.java | Uses CatalogTable builder instead of deprecated CatalogTable.of. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-hudi/pom.xml | Introduces dedicated Flink1-only hadoop compatibility version + Hudi flink major selection. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-elasticsearch/src/test/java/org/apache/flink/cdc/connectors/elasticsearch/sink/ElasticsearchDataSinkITCaseTest.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-elasticsearch/src/test/java/org/apache/flink/cdc/connectors/elasticsearch/sink/Elasticsearch7DataSinkITCaseTest.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-elasticsearch/src/test/java/org/apache/flink/cdc/connectors/elasticsearch/sink/Elasticsearch6DataSinkITCaseTest.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-doris/src/test/java/org/apache/flink/cdc/connectors/doris/sink/DorisPipelineITCase.java | Replaces RestartStrategies with RestartStrategyUtils helper. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-doris/src/test/java/org/apache/flink/cdc/connectors/doris/sink/DorisMetadataApplierITCase.java | Uses EventTypeInfo + env.fromData and RestartStrategyUtils. |
| flink-cdc-connect/flink-cdc-pipeline-connectors/flink-cdc-pipeline-connector-doris/pom.xml | Makes Doris connector artifactId independent of Flink2 build via a dedicated major property. |
| flink-cdc-composer/src/test/java/org/apache/flink/cdc/composer/testsource/source/DistributedSourceFunction.java | Updates SourceFunction.SourceContext references for Flink 2.x API. |
| flink-cdc-composer/src/main/java/org/apache/flink/cdc/composer/flink/translator/DataSinkTranslator.java | Supports new SupportsPreWriteTopology / SupportsPreCommitTopology sink interfaces. |
| .github/workflows/modules.py | Removes obsolete core_2.x module aliasing. |
| .github/workflows/flink_cdc_ci.yml | Adds Flink 2.x unit test jobs for pipeline + source connectors; adjusts E2E matrix. |
💡 Add Copilot custom instructions for smarter, more guided reviews. Learn how to get started.
...-cdc/src/test/java/org/apache/flink/cdc/connectors/mongodb/table/MongoDBConnectorITCase.java
Outdated
Show resolved
Hide resolved
This file contains hidden or bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Sign up for free
to join this conversation on GitHub.
Already have an account?
Sign in to comment
Add this suggestion to a batch that can be applied as a single commit.This suggestion is invalid because no changes were made to the code.Suggestions cannot be applied while the pull request is closed.Suggestions cannot be applied while viewing a subset of changes.Only one suggestion per line can be applied in a batch.Add this suggestion to a batch that can be applied as a single commit.Applying suggestions on deleted lines is not supported.You must change the existing code in this line in order to create a valid suggestion.Outdated suggestions cannot be applied.This suggestion has been applied or marked resolved.Suggestions cannot be applied from pending reviews.Suggestions cannot be applied on multi-line comments.Suggestions cannot be applied while the pull request is queued to merge.Suggestion cannot be applied right now. Please check back later.
Add Flink2 support for Source/Pipeline connector, a follow-up pr of #4294, add
-Pflink2to build project with Flink2 support.Currently, some tests are being skipped due to some difficult to investigate reasons:
Hudi