diff --git a/debezium-core/src/main/java/io/debezium/relational/RelationalSnapshotChangeEventSource.java b/debezium-core/src/main/java/io/debezium/relational/RelationalSnapshotChangeEventSource.java index 3e3f2b61e..1b381db2a 100644 --- a/debezium-core/src/main/java/io/debezium/relational/RelationalSnapshotChangeEventSource.java +++ b/debezium-core/src/main/java/io/debezium/relational/RelationalSnapshotChangeEventSource.java @@ -13,6 +13,7 @@ import java.time.Instant; import java.util.Collection; import java.util.HashMap; +import java.util.ArrayList; import java.util.HashSet; import java.util.Iterator; import java.util.LinkedHashMap; @@ -228,14 +229,22 @@ protected void connectionCreated(RelationalSnapshotContext snapshotContext private Stream toTableIds(Set tableIds, Pattern pattern) { return tableIds .stream() - .filter(tid -> pattern.asPredicate().test(connectorConfig.getTableIdMapper().toString(tid)) || connectorConfig.isSignalDataCollection(tid)) + .filter(tid -> pattern.asPredicate().test(connectorConfig.getTableIdMapper().toString(tid))) .sorted(); } private Set sort(Set capturedTables) throws Exception { String tableIncludeList = connectorConfig.tableIncludeList(); - if (tableIncludeList != null) { - return Strings.listOfRegex(tableIncludeList, Pattern.CASE_INSENSITIVE) + String signalingDataCollection = connectorConfig.getSignalingDataCollectionId(); + List captureTablePatterns = new ArrayList<>(); + if (!Strings.isNullOrBlank(tableIncludeList)) { + captureTablePatterns.addAll(Strings.listOfRegex(tableIncludeList, Pattern.CASE_INSENSITIVE)); + } + if (!Strings.isNullOrBlank(signalingDataCollection)) { + captureTablePatterns.addAll(Strings.listOfRegex(signalingDataCollection, Pattern.CASE_INSENSITIVE)); + } + if (captureTablePatterns.size() > 0) { + return captureTablePatterns .stream() .flatMap(pattern -> toTableIds(capturedTables, pattern)) .collect(Collectors.toCollection(LinkedHashSet::new));