Skip to content

Commit 00b266a

Browse files
rozzagonpinho
andauthored
Source: Added configuration to use the document key for the sourceRecord key
This is potentially a breaking change as the newly added configuration `change.stream.document.key.as.key` defaults to true. Previously, the resume token was used as the source key, but it limits the usefulness of tombstones both for topic compactions and for downstream implementations. Not all events relate to documents (eg drop collection) so fallbacks to resume token for those events. As such this is considered both an improvement and a bug fix. Set to false to revert back to the previous behaviour. KAFKA-360 Co-authored-by: Ross Lawley <[email protected]> Co-authored-by: Goncalo Pinho <[email protected]>
1 parent 358ce92 commit 00b266a

File tree

5 files changed

+44
-6
lines changed

5 files changed

+44
-6
lines changed

CHANGELOG.md

Lines changed: 3 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -6,10 +6,12 @@
66

77
### Improvements
88
- [KAFKA-274](https://jira.mongodb.org/browse/KAFKA-274) Made Debezium (DBZ) ddl events a noop
9+
- [KAFKA-360](https://jira.mongodb.org/browse/KAFKA-360) Added configuration `change.stream.document.key.as.key` and defaults to true.
10+
Previously, the resume token was used as the key, however, that limits the usefulness of tombstones and topic compaction. Set to false to revert.
911

1012
### Bug Fixes
1113
- [KAFKA-378](https://jira.mongodb.org/browse/KAFKA-378) Changed connection uri configuration to password type and for security removed the legacy partition map.
12-
14+
- [KAFKA-360](https://jira.mongodb.org/browse/KAFKA-360) Fixed tombstones on delete by using the `documentKey` if available by default.
1315

1416
## 1.10.0
1517

src/integrationTest/java/com/mongodb/kafka/connect/source/MongoSourceTaskIntegrationTest.java

Lines changed: 9 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -666,7 +666,15 @@ void testSourceEmitsNullValuesOnDelete() {
666666
assertIterableEquals(expectedDocs, actualDocs);
667667

668668
coll.deleteMany(new Document());
669-
getNextResults(task).forEach(s -> assertNull(s.value()));
669+
List<SourceRecord> pollAfterDelete = getNextResults(task);
670+
pollAfterDelete.forEach(s -> assertNull(s.value()));
671+
672+
List<String> documentIds = docs.stream().map(s -> s.get("_id").toString()).collect(toList());
673+
List<String> connectRecordsKeyIds =
674+
pollAfterDelete.stream()
675+
.map(r -> Document.parse(r.key().toString()).get("_id").toString())
676+
.collect(toList());
677+
assertIterableEquals(documentIds, connectRecordsKeyIds);
670678
}
671679
}
672680

src/main/java/com/mongodb/kafka/connect/source/MongoSourceConfig.java

Lines changed: 20 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -289,6 +289,15 @@ public class MongoSourceConfig extends AbstractConfig {
289289
PUBLISH_FULL_DOCUMENT_ONLY_CONFIG,
290290
PUBLISH_FULL_DOCUMENT_ONLY_TOMBSTONE_ON_DELETE_DEFAULT);
291291

292+
public static final String DOCUMENT_KEY_AS_KEY_CONFIG = "change.stream.document.key.as.key";
293+
private static final boolean DOCUMENT_KEY_AS_KEY_DEFAULT = true;
294+
private static final String DOCUMENT_KEY_AS_KEY_DISPLAY =
295+
"Use the `documentKey` for the source record key";
296+
private static final String DOCUMENT_KEY_AS_KEY_DOC =
297+
format(
298+
"Use the document key as the source record key. Defaults to: %s",
299+
DOCUMENT_KEY_AS_KEY_DEFAULT);
300+
292301
public static final String FULL_DOCUMENT_BEFORE_CHANGE_CONFIG =
293302
"change.stream.full.document.before.change";
294303
private static final String FULL_DOCUMENT_BEFORE_CHANGE_DISPLAY =
@@ -1028,6 +1037,17 @@ public Map<String, ConfigValue> validateAll(final Map<String, String> props) {
10281037
Width.MEDIUM,
10291038
PUBLISH_FULL_DOCUMENT_ONLY_TOMBSTONE_ON_DELETE_DISPLAY);
10301039

1040+
configDef.define(
1041+
DOCUMENT_KEY_AS_KEY_CONFIG,
1042+
Type.BOOLEAN,
1043+
DOCUMENT_KEY_AS_KEY_DEFAULT,
1044+
Importance.MEDIUM,
1045+
DOCUMENT_KEY_AS_KEY_DOC,
1046+
group,
1047+
++orderInGroup,
1048+
Width.MEDIUM,
1049+
DOCUMENT_KEY_AS_KEY_DISPLAY);
1050+
10311051
configDef.define(
10321052
FULL_DOCUMENT_BEFORE_CHANGE_CONFIG,
10331053
Type.STRING,

src/main/java/com/mongodb/kafka/connect/source/MongoSourceTask.java

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -89,6 +89,7 @@ public final class MongoSourceTask extends SourceTask {
8989
static final Logger LOGGER = LoggerFactory.getLogger(MongoSourceTask.class);
9090
private static final String CONNECTOR_TYPE = "source";
9191
public static final String ID_FIELD = "_id";
92+
public static final String DOCUMENT_KEY_FIELD = "documentKey";
9293
static final String COPY_KEY = "copy";
9394
private static final String NS_KEY = "ns";
9495
private static final int UNKNOWN_FIELD_ERROR = 40415;

src/main/java/com/mongodb/kafka/connect/source/StartedMongoSourceTask.java

Lines changed: 11 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -20,6 +20,7 @@
2020
import static com.mongodb.kafka.connect.source.MongoSourceConfig.BATCH_SIZE_CONFIG;
2121
import static com.mongodb.kafka.connect.source.MongoSourceConfig.COLLECTION_CONFIG;
2222
import static com.mongodb.kafka.connect.source.MongoSourceConfig.DATABASE_CONFIG;
23+
import static com.mongodb.kafka.connect.source.MongoSourceConfig.DOCUMENT_KEY_AS_KEY_CONFIG;
2324
import static com.mongodb.kafka.connect.source.MongoSourceConfig.HEARTBEAT_INTERVAL_MS_CONFIG;
2425
import static com.mongodb.kafka.connect.source.MongoSourceConfig.HEARTBEAT_TOPIC_NAME_CONFIG;
2526
import static com.mongodb.kafka.connect.source.MongoSourceConfig.POLL_AWAIT_TIME_MS_CONFIG;
@@ -29,6 +30,7 @@
2930
import static com.mongodb.kafka.connect.source.MongoSourceConfig.StartupConfig.StartupMode.COPY_EXISTING;
3031
import static com.mongodb.kafka.connect.source.MongoSourceConfig.StartupConfig.StartupMode.TIMESTAMP;
3132
import static com.mongodb.kafka.connect.source.MongoSourceTask.COPY_KEY;
33+
import static com.mongodb.kafka.connect.source.MongoSourceTask.DOCUMENT_KEY_FIELD;
3234
import static com.mongodb.kafka.connect.source.MongoSourceTask.ID_FIELD;
3335
import static com.mongodb.kafka.connect.source.MongoSourceTask.LOGGER;
3436
import static com.mongodb.kafka.connect.source.MongoSourceTask.createPartitionMap;
@@ -247,10 +249,15 @@ private List<SourceRecord> pollInternal() {
247249
statisticsManager.currentStatistics().getMongodbBytesRead().sample(sizeBytes);
248250
}
249251

250-
BsonDocument keyDocument =
251-
sourceConfig.getKeyOutputFormat() == MongoSourceConfig.OutputFormat.SCHEMA
252-
? changeStreamDocument
253-
: new BsonDocument(ID_FIELD, changeStreamDocument.get(ID_FIELD));
252+
BsonDocument keyDocument;
253+
if (sourceConfig.getKeyOutputFormat() == MongoSourceConfig.OutputFormat.SCHEMA) {
254+
keyDocument = changeStreamDocument;
255+
} else if (sourceConfig.getBoolean(DOCUMENT_KEY_AS_KEY_CONFIG)
256+
&& changeStreamDocument.containsKey(DOCUMENT_KEY_FIELD)) {
257+
keyDocument = changeStreamDocument.getDocument(DOCUMENT_KEY_FIELD);
258+
} else {
259+
keyDocument = new BsonDocument(ID_FIELD, changeStreamDocument.get(ID_FIELD));
260+
}
254261

255262
createSourceRecord(
256263
keySchemaAndValueProducer,

0 commit comments

Comments
 (0)