29
29
import java .util .concurrent .ExecutorService ;
30
30
import java .util .concurrent .ScheduledExecutorService ;
31
31
import java .util .concurrent .TimeUnit ;
32
+ import java .util .concurrent .atomic .AtomicIntegerFieldUpdater ;
32
33
import java .util .concurrent .atomic .AtomicReference ;
33
34
import org .apache .bookkeeper .client .BKException ;
34
35
import org .apache .bookkeeper .client .api .LastConfirmedAndEntry ;
56
57
public class BlobStoreBackedReadHandleImpl implements ReadHandle , OffloadedLedgerHandle {
57
58
private static final Logger log = LoggerFactory .getLogger (BlobStoreBackedReadHandleImpl .class );
58
59
60
+ protected static final AtomicIntegerFieldUpdater <BlobStoreBackedReadHandleImpl > PENDING_READ_UPDATER =
61
+ AtomicIntegerFieldUpdater .newUpdater (BlobStoreBackedReadHandleImpl .class , "pendingRead" );
62
+
59
63
private final long ledgerId ;
60
64
private final OffloadIndexBlock index ;
61
65
private final BackedInputStream inputStream ;
@@ -71,6 +75,8 @@ enum State {
71
75
72
76
private volatile State state = null ;
73
77
78
+ private volatile int pendingRead ;
79
+
74
80
private volatile long lastAccessTimestamp = System .currentTimeMillis ();
75
81
76
82
private BlobStoreBackedReadHandleImpl (long ledgerId , OffloadIndexBlock index ,
@@ -122,9 +128,17 @@ public CompletableFuture<LedgerEntries> readAsync(long firstEntry, long lastEntr
122
128
getId (), firstEntry , lastEntry , (1 + lastEntry - firstEntry ));
123
129
}
124
130
CompletableFuture <LedgerEntries > promise = new CompletableFuture <>();
125
- touch ();
131
+
132
+ // Ledger handles will be only marked idle when "pendingRead" is "0", it is not needed to update
133
+ // "lastAccessTimestamp" if "pendingRead" is larger than "0".
134
+ // Rather than update "lastAccessTimestamp" when starts a reading, updating it when a reading task is finished
135
+ // is better.
136
+ PENDING_READ_UPDATER .incrementAndGet (this );
137
+ promise .whenComplete ((__ , ex ) -> {
138
+ PENDING_READ_UPDATER .decrementAndGet (BlobStoreBackedReadHandleImpl .this );
139
+ lastAccessTimestamp = System .currentTimeMillis ();
140
+ });
126
141
executor .execute (() -> {
127
- touch ();
128
142
if (state == State .Closed ) {
129
143
log .warn ("Reading a closed read handler. Ledger ID: {}, Read range: {}-{}" ,
130
144
ledgerId , firstEntry , lastEntry );
@@ -213,7 +227,6 @@ public CompletableFuture<LedgerEntries> readAsync(long firstEntry, long lastEntr
213
227
}
214
228
215
229
private void seekToEntry (long nextExpectedId ) throws IOException {
216
- touch ();
217
230
Long knownOffset = entryOffsetsCache .getIfPresent (ledgerId , nextExpectedId );
218
231
if (knownOffset != null ) {
219
232
inputStream .seek (knownOffset );
@@ -324,7 +337,8 @@ public long lastAccessTimestamp() {
324
337
return lastAccessTimestamp ;
325
338
}
326
339
327
- private void touch () {
328
- lastAccessTimestamp = System .currentTimeMillis ();
340
+ @ Override
341
+ public int getPendingRead () {
342
+ return PENDING_READ_UPDATER .get (this );
329
343
}
330
344
}
0 commit comments