23
23
import java .util .Collection ;
24
24
import java .util .LinkedList ;
25
25
import java .util .Queue ;
26
+ import java .util .PriorityQueue ;
26
27
import java .util .concurrent .BlockingQueue ;
27
28
import java .util .concurrent .ExecutorService ;
28
29
import java .util .concurrent .LinkedBlockingQueue ;
29
- import java .util .concurrent .PriorityBlockingQueue ;
30
30
import java .util .concurrent .ScheduledExecutorService ;
31
31
import java .util .concurrent .TimeUnit ;
32
32
import java .util .concurrent .atomic .AtomicBoolean ;
@@ -71,7 +71,7 @@ public class SlidingWindowService {
71
71
/**
72
72
* Blocks that are being written.
73
73
*/
74
- private final Queue <Long > writingBlocks = new PriorityBlockingQueue <>();
74
+ private final Queue <Long > writingBlocks = new PriorityQueue <>();
75
75
/**
76
76
* Whether the service is initialized.
77
77
* After the service is initialized, data in {@link #windowCoreData} is valid.
@@ -106,6 +106,11 @@ public class SlidingWindowService {
106
106
*/
107
107
private volatile long lastWriteTimeNanos = 0 ;
108
108
109
+ /**
110
+ * The maximum offset currently written into writeBlocks.*
111
+ */
112
+ private long maxWriteBlockOffset = 0 ;
113
+
109
114
public SlidingWindowService (WALChannel walChannel , int ioThreadNums , long upperLimit , long scaleUnit ,
110
115
long blockSoftLimit , int writeRateLimit , WALHeaderFlusher flusher ) {
111
116
this .walChannel = walChannel ;
@@ -323,6 +328,7 @@ private BlockBatch pollBlocksLocked() {
323
328
324
329
BlockBatch blockBatch = new BlockBatch (blocks );
325
330
writingBlocks .add (blockBatch .startOffset ());
331
+ maxWriteBlockOffset = blockBatch .endOffset ();
326
332
327
333
return blockBatch ;
328
334
}
@@ -331,10 +337,22 @@ private BlockBatch pollBlocksLocked() {
331
337
* Finish the given block batch, and return the start offset of the first block which has not been flushed yet.
332
338
*/
333
339
private long wroteBlocks (BlockBatch wroteBlocks ) {
340
+ this .pollBlockLock .lock ();
341
+ try {
342
+ return wroteBlocksLocked (wroteBlocks );
343
+ } finally {
344
+ this .pollBlockLock .unlock ();
345
+ }
346
+ }
347
+ /**
348
+ * Finish the given block batch, and return the start offset of the first block which has not been flushed yet.
349
+ * Note: this method is NOT thread safe, and it should be called with {@link #pollBlockLock} locked.
350
+ */
351
+ private long wroteBlocksLocked (BlockBatch wroteBlocks ) {
334
352
boolean removed = writingBlocks .remove (wroteBlocks .startOffset ());
335
353
assert removed ;
336
354
if (writingBlocks .isEmpty ()) {
337
- return wroteBlocks . startOffset () + WALUtil . alignLargeByBlockSize ( wroteBlocks . blockBatchSize ()) ;
355
+ return this . maxWriteBlockOffset ;
338
356
}
339
357
return writingBlocks .peek ();
340
358
}
0 commit comments