Skip to content

Commit

Permalink
DSCON-293: Fixed CDC event timestamp not propagating properly
Browse files Browse the repository at this point in the history
  • Loading branch information
Mallik Ankati committed May 7, 2020
1 parent 274e969 commit 5481902
Show file tree
Hide file tree
Showing 2 changed files with 28 additions and 8 deletions.
Expand Up @@ -146,6 +146,10 @@ public void getChangesForTables(ChangeTable[] changeTables, Lsn intervalFromLsn,
final Lsn fromLsn = changeTable.getStartLsn().compareTo(intervalFromLsn) > 0 ? changeTable.getStartLsn() : intervalFromLsn;
LOGGER.trace("Getting changes for table {} in range[{}, {}]", changeTable, fromLsn, intervalToLsn);
preparers[idx] = statement -> {
String fetchSizeStr = config().asProperties().getProperty("incremental.fetch.size");
if (fetchSizeStr != null && fetchSizeStr.trim().length() > 0) {
statement.setFetchSize(Integer.parseInt(fetchSizeStr));
}
statement.setBytes(1, fromLsn.getBinary());
statement.setBytes(2, intervalToLsn.getBinary());
};
Expand Down
Expand Up @@ -5,8 +5,10 @@
*/
package io.debezium.relational;

import java.time.Instant;
import java.util.Objects;

import io.debezium.data.Envelope;
import org.apache.kafka.connect.data.Struct;

import io.debezium.data.Envelope.Operation;
Expand Down Expand Up @@ -63,7 +65,8 @@ private void emitCreateRecord(Receiver receiver, TableSchema tableSchema)
Object[] newColumnValues = getNewColumnValues();
Object newKey = tableSchema.keyFromColumnData(newColumnValues);
Struct newValue = tableSchema.valueFromColumnData(newColumnValues);
Struct envelope = tableSchema.getEnvelopeSchema().create(newValue, offsetContext.getSourceInfo(), clock.currentTimeInMillis());
long timestamp = getTimestampFrom(offsetContext);
Struct envelope = tableSchema.getEnvelopeSchema().create(newValue, offsetContext.getSourceInfo(), timestamp);

receiver.changeRecord(tableSchema, Operation.CREATE, newKey, envelope, offsetContext);
}
Expand All @@ -73,7 +76,8 @@ private void emitReadRecord(Receiver receiver, TableSchema tableSchema)
Object[] newColumnValues = getNewColumnValues();
Object newKey = tableSchema.keyFromColumnData(newColumnValues);
Struct newValue = tableSchema.valueFromColumnData(newColumnValues);
Struct envelope = tableSchema.getEnvelopeSchema().read(newValue, offsetContext.getSourceInfo(), clock.currentTimeInMillis());
long timestamp = getTimestampFrom(offsetContext);
Struct envelope = tableSchema.getEnvelopeSchema().read(newValue, offsetContext.getSourceInfo(), timestamp);

receiver.changeRecord(tableSchema, Operation.READ, newKey, envelope, offsetContext);
}
Expand All @@ -88,18 +92,18 @@ private void emitUpdateRecord(Receiver receiver, TableSchema tableSchema)

Struct newValue = tableSchema.valueFromColumnData(newColumnValues);
Struct oldValue = tableSchema.valueFromColumnData(oldColumnValues);

long timestamp = getTimestampFrom(offsetContext);
// regular update
if (Objects.equals(oldKey, newKey)) {
Struct envelope = tableSchema.getEnvelopeSchema().update(oldValue, newValue, offsetContext.getSourceInfo(), clock.currentTimeInMillis());
Struct envelope = tableSchema.getEnvelopeSchema().update(oldValue, newValue, offsetContext.getSourceInfo(), timestamp);
receiver.changeRecord(tableSchema, Operation.UPDATE, newKey, envelope, offsetContext);
}
// PK update -> emit as delete and re-insert with new key
else {
Struct envelope = tableSchema.getEnvelopeSchema().delete(oldValue, offsetContext.getSourceInfo(), clock.currentTimeInMillis());
Struct envelope = tableSchema.getEnvelopeSchema().delete(oldValue, offsetContext.getSourceInfo(), timestamp);
receiver.changeRecord(tableSchema, Operation.DELETE, oldKey, envelope, offsetContext);

envelope = tableSchema.getEnvelopeSchema().create(newValue, offsetContext.getSourceInfo(), clock.currentTimeInMillis());
envelope = tableSchema.getEnvelopeSchema().create(newValue, offsetContext.getSourceInfo(), timestamp);
receiver.changeRecord(tableSchema, Operation.CREATE, newKey, envelope, offsetContext);
}
}
Expand All @@ -108,8 +112,8 @@ private void emitDeleteRecord(Receiver receiver, TableSchema tableSchema) throws
Object[] oldColumnValues = getOldColumnValues();
Object oldKey = tableSchema.keyFromColumnData(oldColumnValues);
Struct oldValue = tableSchema.valueFromColumnData(oldColumnValues);

Struct envelope = tableSchema.getEnvelopeSchema().delete(oldValue, offsetContext.getSourceInfo(), clock.currentTimeInMillis());
long timestamp = getTimestampFrom(offsetContext);
Struct envelope = tableSchema.getEnvelopeSchema().delete(oldValue, offsetContext.getSourceInfo(), timestamp);
receiver.changeRecord(tableSchema, Operation.DELETE, oldKey, envelope, offsetContext);
}

Expand All @@ -127,4 +131,16 @@ private void emitDeleteRecord(Receiver receiver, TableSchema tableSchema) throws
* Returns the new row state in case of a CREATE or READ.
*/
protected abstract Object[] getNewColumnValues();

protected long getTimestampFrom(OffsetContext offsetContext){
long timestamp = clock.currentTimeInMillis();
Object timeObj = offsetContext.getSourceInfo().get(Envelope.FieldName.TIMESTAMP);
if (timeObj != null && timeObj instanceof Long) {
timestamp = (Long) timeObj;
} else if (timeObj != null && timeObj instanceof Instant){
Instant instant = (Instant) offsetContext.getSourceInfo().get(Envelope.FieldName.TIMESTAMP);
timestamp = instant.toEpochMilli();
}
return timestamp;
}
}

0 comments on commit 5481902

Please sign in to comment.