58
58
import org .apache .flink .streaming .connectors .pulsar .internal .TopicSubscription ;
59
59
import org .apache .flink .streaming .connectors .pulsar .internal .TopicSubscriptionSerializer ;
60
60
import org .apache .flink .streaming .connectors .pulsar .serialization .PulsarDeserializationSchema ;
61
+ import org .apache .flink .streaming .connectors .pulsar .table .PulsarTableOptions ;
61
62
import org .apache .flink .streaming .runtime .operators .util .AssignerWithPeriodicWatermarksAdapter ;
62
63
import org .apache .flink .streaming .runtime .operators .util .AssignerWithPunctuatedWatermarksAdapter ;
63
64
import org .apache .flink .streaming .runtime .tasks .ProcessingTimeService ;
64
65
import org .apache .flink .util .ExceptionUtils ;
65
66
import org .apache .flink .util .SerializedValue ;
67
+ import org .apache .flink .util .TimeUtils ;
66
68
67
69
import org .apache .flink .shaded .guava18 .com .google .common .collect .Sets ;
68
70
73
75
import org .apache .pulsar .shade .com .google .common .collect .Maps ;
74
76
import org .apache .pulsar .shade .org .apache .commons .lang3 .StringUtils ;
75
77
78
+ import java .time .Duration ;
76
79
import java .util .ArrayList ;
77
80
import java .util .HashMap ;
78
81
import java .util .HashSet ;
85
88
import java .util .Set ;
86
89
import java .util .TreeMap ;
87
90
import java .util .UUID ;
91
+ import java .util .concurrent .Executors ;
92
+ import java .util .concurrent .ScheduledExecutorService ;
93
+ import java .util .concurrent .TimeUnit ;
88
94
import java .util .concurrent .atomic .AtomicReference ;
89
95
import java .util .stream .Collectors ;
90
96
@@ -238,6 +244,12 @@ public class FlinkPulsarSource<T> extends RichParallelSourceFunction<T>
238
244
239
245
private long startupOffsetsTimestamp = -1L ;
240
246
247
+ private boolean enableOffsetAutoCommit ;
248
+
249
+ private Duration offsetAutoCommitInterval ;
250
+
251
+ private ScheduledExecutorService offsetCommitScheduler ;
252
+
241
253
public FlinkPulsarSource (
242
254
String adminUrl ,
243
255
ClientConfigurationData clientConf ,
@@ -264,6 +276,23 @@ public FlinkPulsarSource(
264
276
}
265
277
this .oldStateVersion =
266
278
SourceSinkUtils .getOldStateVersion (caseInsensitiveParams , oldStateVersion );
279
+
280
+ this .enableOffsetAutoCommit =
281
+ Boolean .parseBoolean (
282
+ properties
283
+ .getOrDefault (
284
+ PulsarTableOptions .ENABLE_OFFSET_AUTO_COMMIT .key (), "true" )
285
+ .toString ());
286
+ if (enableOffsetAutoCommit ) {
287
+ log .info ("enable offset auto commit." );
288
+ this .offsetAutoCommitInterval =
289
+ TimeUtils .parseDuration (
290
+ properties
291
+ .getOrDefault (
292
+ PulsarTableOptions .OFFSET_AUTO_COMMIT_INTERVAL .key (),
293
+ "60 s" )
294
+ .toString ());
295
+ }
267
296
}
268
297
269
298
public FlinkPulsarSource (
@@ -551,6 +580,28 @@ public void open(Configuration parameters) throws Exception {
551
580
ownedTopicStarts );
552
581
}
553
582
}
583
+
584
+ if (!((StreamingRuntimeContext ) getRuntimeContext ()).isCheckpointingEnabled ()
585
+ && enableOffsetAutoCommit ) {
586
+ this .offsetCommitScheduler = Executors .newScheduledThreadPool (1 );
587
+ this .offsetCommitScheduler .scheduleAtFixedRate (
588
+ () -> {
589
+ if (pulsarFetcher != null ) {
590
+ Map <TopicRange , MessageId > consumedOffsets =
591
+ pulsarFetcher .snapshotCurrentState ();
592
+ try {
593
+ log .info ("commit offset to pulsar cluster." );
594
+ pulsarFetcher .commitOffsetToPulsar (
595
+ consumedOffsets , offsetCommitCallback );
596
+ } catch (InterruptedException e ) {
597
+
598
+ }
599
+ }
600
+ },
601
+ 0 ,
602
+ offsetAutoCommitInterval != null ? offsetAutoCommitInterval .getSeconds () : 60 ,
603
+ TimeUnit .SECONDS );
604
+ }
554
605
}
555
606
556
607
protected String getSubscriptionName () {
@@ -742,6 +793,10 @@ public void close() throws Exception {
742
793
}
743
794
}
744
795
796
+ if (offsetCommitScheduler != null ) {
797
+ offsetCommitScheduler .shutdown ();
798
+ }
799
+
745
800
try {
746
801
super .close ();
747
802
} catch (Exception e ) {
0 commit comments