|
79 | 79 | import static org.apache.flink.cdc.connectors.mysql.source.MySqlDataSourceOptions.INCLUDE_COMMENTS_ENABLED;
|
80 | 80 | import static org.apache.flink.cdc.connectors.mysql.source.MySqlDataSourceOptions.PASSWORD;
|
81 | 81 | import static org.apache.flink.cdc.connectors.mysql.source.MySqlDataSourceOptions.PORT;
|
| 82 | +import static org.apache.flink.cdc.connectors.mysql.source.MySqlDataSourceOptions.SCAN_BINLOG_NEWLY_ADDED_TABLE_ENABLED; |
82 | 83 | import static org.apache.flink.cdc.connectors.mysql.source.MySqlDataSourceOptions.SCHEMA_CHANGE_ENABLED;
|
83 | 84 | import static org.apache.flink.cdc.connectors.mysql.source.MySqlDataSourceOptions.SERVER_TIME_ZONE;
|
84 | 85 | import static org.apache.flink.cdc.connectors.mysql.source.MySqlDataSourceOptions.TABLES;
|
@@ -1076,6 +1077,75 @@ public void testIncludeComments() throws Exception {
|
1076 | 1077 | actual.stream().map(Object::toString).collect(Collectors.toList()));
|
1077 | 1078 | }
|
1078 | 1079 |
|
| 1080 | + @Test |
| 1081 | + public void testIncludeCommentsForScanBinlogNewlyAddedTableEnabled() throws Exception { |
| 1082 | + env.setParallelism(1); |
| 1083 | + inventoryDatabase.createAndInitialize(); |
| 1084 | + TableId tableId = TableId.tableId(inventoryDatabase.getDatabaseName(), "products"); |
| 1085 | + TableId newTableId = |
| 1086 | + TableId.tableId(inventoryDatabase.getDatabaseName(), "products_with_comments2"); |
| 1087 | + |
| 1088 | + Map<String, String> options = new HashMap<>(); |
| 1089 | + options.put(HOSTNAME.key(), MYSQL8_CONTAINER.getHost()); |
| 1090 | + options.put(PORT.key(), String.valueOf(MYSQL8_CONTAINER.getDatabasePort())); |
| 1091 | + options.put(USERNAME.key(), TEST_USER); |
| 1092 | + options.put(PASSWORD.key(), TEST_PASSWORD); |
| 1093 | + options.put(SERVER_TIME_ZONE.key(), "UTC"); |
| 1094 | + options.put(INCLUDE_COMMENTS_ENABLED.key(), "true"); |
| 1095 | + options.put(SCAN_BINLOG_NEWLY_ADDED_TABLE_ENABLED.key(), "true"); |
| 1096 | + options.put(TABLES.key(), inventoryDatabase.getDatabaseName() + ".products\\.*"); |
| 1097 | + Factory.Context context = |
| 1098 | + new FactoryHelper.DefaultContext( |
| 1099 | + Configuration.fromMap(options), null, this.getClass().getClassLoader()); |
| 1100 | + |
| 1101 | + MySqlDataSourceFactory factory = new MySqlDataSourceFactory(); |
| 1102 | + MySqlDataSource dataSource = (MySqlDataSource) factory.createDataSource(context); |
| 1103 | + FlinkSourceProvider sourceProvider = |
| 1104 | + (FlinkSourceProvider) dataSource.getEventSourceProvider(); |
| 1105 | + |
| 1106 | + CloseableIterator<Event> events = |
| 1107 | + env.fromSource( |
| 1108 | + sourceProvider.getSource(), |
| 1109 | + WatermarkStrategy.noWatermarks(), |
| 1110 | + MySqlDataSourceFactory.IDENTIFIER, |
| 1111 | + new EventTypeInfo()) |
| 1112 | + .executeAndCollect(); |
| 1113 | + Thread.sleep(5_000); |
| 1114 | + |
| 1115 | + String createTableSql = |
| 1116 | + String.format( |
| 1117 | + "CREATE TABLE IF NOT EXISTS `%s`.`%s` (\n" |
| 1118 | + + " id INTEGER NOT NULL AUTO_INCREMENT COMMENT 'column comment of id' PRIMARY KEY,\n" |
| 1119 | + + " name VARCHAR(255) NOT NULL DEFAULT 'flink' COMMENT 'column comment of name',\n" |
| 1120 | + + " weight FLOAT COMMENT 'column comment of weight'\n" |
| 1121 | + + ")\n" |
| 1122 | + + "COMMENT 'table comment of products';", |
| 1123 | + inventoryDatabase.getDatabaseName(), "products_with_comments2"); |
| 1124 | + executeSql(inventoryDatabase, createTableSql); |
| 1125 | + |
| 1126 | + // add some column |
| 1127 | + String addColumnSql = |
| 1128 | + String.format( |
| 1129 | + "ALTER TABLE `%s`.`products_with_comments2` ADD COLUMN `description` VARCHAR(512) comment 'column comment of description';", |
| 1130 | + inventoryDatabase.getDatabaseName()); |
| 1131 | + executeSql(inventoryDatabase, addColumnSql); |
| 1132 | + |
| 1133 | + List<Event> expectedEvents = new ArrayList<>(); |
| 1134 | + CreateTableEvent productCreateTableEvent = getProductsCreateTableEvent(tableId); |
| 1135 | + expectedEvents.add(productCreateTableEvent); |
| 1136 | + // generate snapshot data |
| 1137 | + List<Event> productExpectedSnapshot = getSnapshotExpected(tableId); |
| 1138 | + expectedEvents.addAll(productExpectedSnapshot); |
| 1139 | + |
| 1140 | + List<Event> newTableExpectedEvents = getEventsWithComments(newTableId); |
| 1141 | + expectedEvents.addAll(newTableExpectedEvents); |
| 1142 | + |
| 1143 | + List<Event> actual = fetchResults(events, expectedEvents.size()); |
| 1144 | + assertEqualsInAnyOrder( |
| 1145 | + expectedEvents.stream().map(Object::toString).collect(Collectors.toList()), |
| 1146 | + actual.stream().map(Object::toString).collect(Collectors.toList())); |
| 1147 | + } |
| 1148 | + |
1079 | 1149 | private void executeSql(UniqueDatabase database, String sql) throws SQLException {
|
1080 | 1150 | try (Connection connection = database.getJdbcConnection();
|
1081 | 1151 | Statement statement = connection.createStatement()) {
|
|
0 commit comments