Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

[FLINK-37191][mysql] Avoid back filling if the lowWatermark of BinlogSplit is equal to highWatermark. #3902

Merged
merged 4 commits into from
Feb 11, 2025
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -292,7 +292,7 @@ private boolean shouldEmit(SourceRecord sourceRecord) {
private boolean hasEnterPureBinlogPhase(TableId tableId, BinlogOffset position) {
// the existed tables those have finished snapshot reading
if (maxSplitHighWatermarkMap.containsKey(tableId)
&& position.isAtOrAfter(maxSplitHighWatermarkMap.get(tableId))) {
&& position.isAfter(maxSplitHighWatermarkMap.get(tableId))) {
pureBinlogPhaseTables.add(tableId);
return true;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -154,11 +154,6 @@ protected SnapshotResult<MySqlOffsetContext> doExecute(
hooks.getPreLowWatermarkAction().accept(jdbcConnection, snapshotSplit);
}
final BinlogOffset lowWatermark = DebeziumUtils.currentBinlogOffset(jdbcConnection);
lowWatermark
.getOffset()
.put(
BinlogOffset.TIMESTAMP_KEY,
String.valueOf(clock.currentTime().getEpochSecond()));
LOG.info(
"Snapshot step 1 - Determining low watermark {} for split {}",
lowWatermark,
Expand Down Expand Up @@ -192,11 +187,6 @@ protected SnapshotResult<MySqlOffsetContext> doExecute(
} else {
// Get the current binlog offset as HW
highWatermark = DebeziumUtils.currentBinlogOffset(jdbcConnection);
highWatermark
.getOffset()
.put(
BinlogOffset.TIMESTAMP_KEY,
String.valueOf(clock.currentTime().getEpochSecond()));
}

LOG.info(
Expand Down
Copy link
Contributor

@yuxiqian yuxiqian Feb 6, 2025

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

BinlogOffset#compareTo is a general-purposed comparison method, and this change might affect more than backfill necessity check. Is it possible to tweak SnapshotSplitReader#isBackfillRequired only?

Original file line number Diff line number Diff line change
Expand Up @@ -232,7 +232,9 @@ public int compareTo(BinlogOffset that) {
// compared ...
long timestamp = this.getTimestampSec();
long targetTimestamp = that.getTimestampSec();
return Long.compare(timestamp, targetTimestamp);
if (timestamp != 0 && targetTimestamp != 0) {
return Long.compare(timestamp, targetTimestamp);
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

This change is necessary because that if the TimestampSec is 0, the assumption of These are from different servers, and their binlog coordinates are not related is not correct, these two BinlogOffsets could come from the same server.

}
}

// First compare the MySQL binlog filenames
Expand All @@ -251,12 +253,7 @@ public int compareTo(BinlogOffset that) {
}

// The completed events are the same, so compare the row number ...
if (this.getRestartSkipRows() != that.getRestartSkipRows()) {
return Long.compare(this.getRestartSkipRows(), that.getRestartSkipRows());
}

// The skip rows are the same, so compare the timestamp ...
return Long.compare(this.getTimestampSec(), that.getTimestampSec());
return Long.compare(this.getRestartSkipRows(), that.getRestartSkipRows());
}

public boolean isAtOrBefore(BinlogOffset that) {
Expand Down
Loading