lzshlzsh commented on code in PR #3845:
URL: https://github.com/apache/flink-cdc/pull/3845#discussion_r2303851624


##########
flink-cdc-connect/flink-cdc-source-connectors/flink-connector-mysql-cdc/src/main/java/io/debezium/connector/mysql/GtidUtils.java:
##########
@@ -36,36 +38,64 @@ public class GtidUtils {
     public static GtidSet fixRestoredGtidSet(GtidSet serverGtidSet, GtidSet 
restoredGtidSet) {
         Map<String, GtidSet.UUIDSet> newSet = new HashMap<>();
         serverGtidSet.getUUIDSets().forEach(uuidSet -> 
newSet.put(uuidSet.getUUID(), uuidSet));
-        for (GtidSet.UUIDSet uuidSet : restoredGtidSet.getUUIDSets()) {
-            GtidSet.UUIDSet serverUuidSet = newSet.get(uuidSet.getUUID());
+        for (GtidSet.UUIDSet restoredUuidSet : restoredGtidSet.getUUIDSets()) {
+            GtidSet.UUIDSet serverUuidSet = 
newSet.get(restoredUuidSet.getUUID());
             if (serverUuidSet != null) {
-                long restoredIntervalEnd = getIntervalEnd(uuidSet);
-                List<com.github.shyiko.mysql.binlog.GtidSet.Interval> 
newIntervals =
-                        new ArrayList<>();
-                for (GtidSet.Interval serverInterval : 
serverUuidSet.getIntervals()) {
-                    if (serverInterval.getEnd() <= restoredIntervalEnd) {
-                        newIntervals.add(
-                                new 
com.github.shyiko.mysql.binlog.GtidSet.Interval(
-                                        serverInterval.getStart(), 
serverInterval.getEnd()));
-                    } else if (serverInterval.getStart() <= restoredIntervalEnd
-                            && serverInterval.getEnd() > restoredIntervalEnd) {
-                        newIntervals.add(
+                List<GtidSet.Interval> serverIntervals = 
serverUuidSet.getIntervals();
+                List<GtidSet.Interval> restoredIntervals = 
restoredUuidSet.getIntervals();
+
+                long earliestRestoredTx = 
getMinIntervalStart(restoredIntervals);
+
+                List<com.github.shyiko.mysql.binlog.GtidSet.Interval> merged = 
new ArrayList<>();
+
+                // Process each server interval
+                for (GtidSet.Interval serverInterval : serverIntervals) {
+                    // First, check if any part comes before earliest restored
+                    if (serverInterval.getStart() < earliestRestoredTx) {
+                        long end = Math.min(serverInterval.getEnd(), 
earliestRestoredTx - 1);
+                        merged.add(
                                 new 
com.github.shyiko.mysql.binlog.GtidSet.Interval(
-                                        serverInterval.getStart(), 
restoredIntervalEnd));
+                                        serverInterval.getStart(), end));

Review Comment:
   > > When starting with the 'scan.startup.mode' = 'earliest-offset', after 
reading 182580 and 182579 data, a checkpoint is made. The saved gtid is 
A:182580-182580, "row": "1", "event": "2".
   > 
   > I'm not sure that its 100% correct. Because in case of `earliest-offset`, 
Pipeline should receive all data from the beginning, in this case `A:1-182571` 
and then 182580, 182579, 182578.
   > 
   > So after 182580 tx, saved GTID Set should be `A:1-182571:182580-182580` 
"row": "1", "event": "2". And in this case after restart "merged" version 
should be still `A:1-182571:182580-182580`
   
   I have did a test and get the above result. It seems that 
#MySqlStreamingChangeEventSource update effectiveOffsetContext on GTID, and not 
receive `A:1-182571` that has been purged.
   
   ```
   2025-08-27 20:39:40.394 [blc-127.0.0.1:3308] 
MySqlStreamingChangeEventSource.java:558 DEBUG 
io.debezium.connector.mysql.MySqlStreamingChangeEventSource - GTID transaction: 
Event{header=EventHeaderV4{timestamp=1756127165000, eventType=GTID, 
serverId=123, headerLength=19, dataLength=60, nextPosition=316, flags=0}, 
data=GtidEventData{flags=0, gtid='fc992a75-c2a9-11ee-82e7-0242ac120004:182580'}}
   2025-08-27 20:39:40.401 [blc-127.0.0.1:3308] 
MySqlStreamingChangeEventSource.java:558 DEBUG 
io.debezium.connector.mysql.MySqlStreamingChangeEventSource - GTID transaction: 
Event{header=EventHeaderV4{timestamp=1756127179000, eventType=GTID, 
serverId=123, headerLength=19, dataLength=60, nextPosition=634, flags=0}, 
data=GtidEventData{flags=0, gtid='fc992a75-c2a9-11ee-82e7-0242ac120004:182579'}}
   2025-08-27 20:39:40.401 [blc-127.0.0.1:3308] 
MySqlStreamingChangeEventSource.java:558 DEBUG 
io.debezium.connector.mysql.MySqlStreamingChangeEventSource - GTID transaction: 
Event{header=EventHeaderV4{timestamp=1756127436000, eventType=GTID, 
serverId=123, headerLength=19, dataLength=60, nextPosition=952, flags=0}, 
data=GtidEventData{flags=0, gtid='fc992a75-c2a9-11ee-82e7-0242ac120004:182578'}}
   ```
   
   <img width="1917" height="470" alt="image" 
src="https://github.com/user-attachments/assets/560b2404-50d0-492c-9bf7-bfb98f0c74dc";
 />
   
   



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscr...@flink.apache.org

For queries about this service, please contact Infrastructure at:
us...@infra.apache.org

Reply via email to