Skip to content

Commit 5544ed2

Browse files
committed
PIP-174: New managed ledger entry cache implementation
1 parent 93e947b commit 5544ed2

File tree

26 files changed

+1784
-80
lines changed

26 files changed

+1784
-80
lines changed

conf/broker.conf

+6
Original file line numberDiff line numberDiff line change
@@ -1008,6 +1008,12 @@ managedLedgerCacheSizeMB=
10081008
# Whether we should make a copy of the entry payloads when inserting in cache
10091009
managedLedgerCacheCopyEntries=false
10101010

1011+
# The class name for the implementation of ManagedLedger cache manager component.
1012+
# Options are:
1013+
# - org.apache.bookkeeper.mledger.impl.cache.SharedEntryCacheManagerImpl
1014+
# - org.apache.bookkeeper.mledger.impl.cache.RangeEntryCacheManagerImpl
1015+
managedLedgerCacheManagerImplementationClass=org.apache.bookkeeper.mledger.impl.cache.SharedEntryCacheManagerImpl
1016+
10111017
# Threshold to which bring down the cache level when eviction is triggered
10121018
managedLedgerCacheEvictionWatermark=0.9
10131019

conf/standalone.conf

+6
Original file line numberDiff line numberDiff line change
@@ -680,6 +680,12 @@ managedLedgerCacheSizeMB=
680680
# Whether we should make a copy of the entry payloads when inserting in cache
681681
managedLedgerCacheCopyEntries=false
682682

683+
# The class name for the implementation of ManagedLedger cache manager component.
684+
# Options are:
685+
# - org.apache.bookkeeper.mledger.impl.cache.SharedEntryCacheManagerImpl
686+
# - org.apache.bookkeeper.mledger.impl.cache.RangeEntryCacheManagerImpl
687+
managedLedgerCacheManagerImplementationClass=org.apache.bookkeeper.mledger.impl.cache.SharedEntryCacheManagerImpl
688+
683689
# Threshold to which bring down the cache level when eviction is triggered
684690
managedLedgerCacheEvictionWatermark=0.9
685691

managed-ledger/src/main/java/org/apache/bookkeeper/mledger/ManagedLedgerFactoryConfig.java

+12
Original file line numberDiff line numberDiff line change
@@ -21,6 +21,7 @@
2121
import lombok.Data;
2222
import org.apache.bookkeeper.common.annotation.InterfaceAudience;
2323
import org.apache.bookkeeper.common.annotation.InterfaceStability;
24+
import org.apache.bookkeeper.mledger.impl.cache.SharedEntryCacheManagerImpl;
2425
import org.apache.bookkeeper.mledger.proto.MLDataFormats;
2526

2627
/**
@@ -91,4 +92,15 @@ public class ManagedLedgerFactoryConfig {
9192
* ManagedCursorInfo compression type. If the compression type is null or invalid, don't compress data.
9293
*/
9394
private String managedCursorInfoCompressionType = MLDataFormats.CompressionType.NONE.name();
95+
96+
/**
97+
* Class name for the implementation of {@link org.apache.bookkeeper.mledger.impl.cache.EntryCacheManager}.
98+
*
99+
* Options are:
100+
* <ul>
101+
* <li>{@link org.apache.bookkeeper.mledger.impl.cache.SharedEntryCacheManagerImpl}</li>
102+
* <li>{@link org.apache.bookkeeper.mledger.impl.cache.RangeEntryCacheManagerImpl}</li>
103+
* </ul>
104+
*/
105+
private String entryCacheManagerClassName = SharedEntryCacheManagerImpl.class.getName();
94106
}

managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/ManagedLedgerFactoryImpl.java

+8-4
Original file line numberDiff line numberDiff line change
@@ -69,7 +69,6 @@
6969
import org.apache.bookkeeper.mledger.impl.ManagedLedgerImpl.State;
7070
import org.apache.bookkeeper.mledger.impl.MetaStore.MetaStoreCallback;
7171
import org.apache.bookkeeper.mledger.impl.cache.EntryCacheManager;
72-
import org.apache.bookkeeper.mledger.impl.cache.RangeEntryCacheManagerImpl;
7372
import org.apache.bookkeeper.mledger.proto.MLDataFormats;
7473
import org.apache.bookkeeper.mledger.proto.MLDataFormats.LongProperty;
7574
import org.apache.bookkeeper.mledger.proto.MLDataFormats.ManagedCursorInfo;
@@ -193,7 +192,11 @@ private ManagedLedgerFactoryImpl(MetadataStoreExtended metadataStore,
193192
config.getManagedCursorInfoCompressionType());
194193
this.config = config;
195194
this.mbean = new ManagedLedgerFactoryMBeanImpl(this);
196-
this.entryCacheManager = new RangeEntryCacheManagerImpl(this);
195+
196+
Class<EntryCacheManager> ecmClass =
197+
(Class<EntryCacheManager>) Class.forName(config.getEntryCacheManagerClassName());
198+
this.entryCacheManager = ecmClass.getDeclaredConstructor(ManagedLedgerFactoryImpl.class).newInstance(this);
199+
197200
this.statsTask = scheduledExecutor.scheduleWithFixedDelay(catchingAndLoggingThrowables(this::refreshStats),
198201
0, config.getStatsPeriodSeconds(), TimeUnit.SECONDS);
199202
this.flushCursorsTask = scheduledExecutor.scheduleAtFixedRate(catchingAndLoggingThrowables(this::flushCursors),
@@ -592,7 +595,8 @@ public void closeFailed(ManagedLedgerException exception, Object ctx) {
592595
}));
593596
}
594597
}));
595-
entryCacheManager.clear();
598+
599+
entryCacheManager.close();
596600
return FutureUtil.waitForAll(futures).thenAccept(__ -> {
597601
//wait for tasks in scheduledExecutor executed.
598602
scheduledExecutor.shutdown();
@@ -653,7 +657,7 @@ public void closeFailed(ManagedLedgerException exception, Object ctx) {
653657

654658
scheduledExecutor.shutdownNow();
655659

656-
entryCacheManager.clear();
660+
entryCacheManager.close();
657661
}
658662

659663
@Override

managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/cache/EntryCacheDisabled.java

+2-2
Original file line numberDiff line numberDiff line change
@@ -87,7 +87,7 @@ public void asyncReadEntry(ReadHandle lh, long firstEntry, long lastEntry, boole
8787
try {
8888
for (LedgerEntry e : ledgerEntries) {
8989
// Insert the entries at the end of the list (they will be unsorted for now)
90-
EntryImpl entry = RangeEntryCacheManagerImpl.create(e, interceptor);
90+
EntryImpl entry = EntryCacheManager.create(e, interceptor);
9191
entries.add(entry);
9292
totalSize += entry.getLength();
9393
}
@@ -119,7 +119,7 @@ public void asyncReadEntry(ReadHandle lh, PositionImpl position, AsyncCallbacks.
119119
Iterator<LedgerEntry> iterator = ledgerEntries.iterator();
120120
if (iterator.hasNext()) {
121121
LedgerEntry ledgerEntry = iterator.next();
122-
EntryImpl returnEntry = RangeEntryCacheManagerImpl.create(ledgerEntry, interceptor);
122+
EntryImpl returnEntry = EntryCacheManager.create(ledgerEntry, interceptor);
123123

124124
ml.getFactory().getMbean().recordCacheMiss(1, returnEntry.getLength());
125125
ml.getMbean().addReadEntriesSample(1, returnEntry.getLength());

managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/cache/EntryCacheManager.java

+35-1
Original file line numberDiff line numberDiff line change
@@ -18,9 +18,15 @@
1818
*/
1919
package org.apache.bookkeeper.mledger.impl.cache;
2020

21+
import io.netty.buffer.ByteBuf;
22+
import org.apache.bookkeeper.client.api.LedgerEntry;
23+
import org.apache.bookkeeper.client.impl.LedgerEntryImpl;
24+
import org.apache.bookkeeper.mledger.Entry;
25+
import org.apache.bookkeeper.mledger.impl.EntryImpl;
2126
import org.apache.bookkeeper.mledger.impl.ManagedLedgerImpl;
27+
import org.apache.bookkeeper.mledger.intercept.ManagedLedgerInterceptor;
2228

23-
public interface EntryCacheManager {
29+
public interface EntryCacheManager extends AutoCloseable {
2430
EntryCache getEntryCache(ManagedLedgerImpl ml);
2531

2632
void removeEntryCache(String name);
@@ -36,4 +42,32 @@ public interface EntryCacheManager {
3642
void updateCacheEvictionWatermark(double cacheEvictionWatermark);
3743

3844
double getCacheEvictionWatermark();
45+
46+
static Entry create(long ledgerId, long entryId, ByteBuf data) {
47+
return EntryImpl.create(ledgerId, entryId, data);
48+
}
49+
50+
static EntryImpl create(LedgerEntry ledgerEntry, ManagedLedgerInterceptor interceptor) {
51+
ManagedLedgerInterceptor.PayloadProcessorHandle processorHandle = null;
52+
if (interceptor != null) {
53+
ByteBuf duplicateBuffer = ledgerEntry.getEntryBuffer().retainedDuplicate();
54+
processorHandle = interceptor
55+
.processPayloadBeforeEntryCache(duplicateBuffer);
56+
if (processorHandle != null) {
57+
ledgerEntry = LedgerEntryImpl.create(ledgerEntry.getLedgerId(), ledgerEntry.getEntryId(),
58+
ledgerEntry.getLength(), processorHandle.getProcessedPayload());
59+
} else {
60+
duplicateBuffer.release();
61+
}
62+
}
63+
EntryImpl returnEntry = EntryImpl.create(ledgerEntry);
64+
if (processorHandle != null) {
65+
processorHandle.release();
66+
ledgerEntry.close();
67+
}
68+
return returnEntry;
69+
}
70+
71+
@Override
72+
void close();
3973
}

managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/cache/RangeEntryCacheImpl.java

+3-3
Original file line numberDiff line numberDiff line change
@@ -50,7 +50,7 @@ public class RangeEntryCacheImpl implements EntryCache {
5050

5151
private final RangeEntryCacheManagerImpl manager;
5252
private final ManagedLedgerImpl ml;
53-
private ManagedLedgerInterceptor interceptor;
53+
private final ManagedLedgerInterceptor interceptor;
5454
private final RangeCache<PositionImpl, EntryImpl> entries;
5555
private final boolean copyEntries;
5656

@@ -221,7 +221,7 @@ private void asyncReadEntry0(ReadHandle lh, PositionImpl position, final ReadEnt
221221
Iterator<LedgerEntry> iterator = ledgerEntries.iterator();
222222
if (iterator.hasNext()) {
223223
LedgerEntry ledgerEntry = iterator.next();
224-
EntryImpl returnEntry = RangeEntryCacheManagerImpl.create(ledgerEntry, interceptor);
224+
EntryImpl returnEntry = EntryCacheManager.create(ledgerEntry, interceptor);
225225

226226
manager.mlFactoryMBean.recordCacheMiss(1, returnEntry.getLength());
227227
ml.getMbean().addReadEntriesSample(1, returnEntry.getLength());
@@ -306,7 +306,7 @@ private void asyncReadEntry0(ReadHandle lh, long firstEntry, long lastEntry, boo
306306
long totalSize = 0;
307307
final List<EntryImpl> entriesToReturn = Lists.newArrayListWithExpectedSize(entriesToRead);
308308
for (LedgerEntry e : ledgerEntries) {
309-
EntryImpl entry = RangeEntryCacheManagerImpl.create(e, interceptor);
309+
EntryImpl entry = EntryCacheManager.create(e, interceptor);
310310
entriesToReturn.add(entry);
311311
totalSize += entry.getLength();
312312
if (shouldCacheEntry) {

managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/cache/RangeEntryCacheManagerImpl.java

+3-29
Original file line numberDiff line numberDiff line change
@@ -21,19 +21,13 @@
2121
import static org.apache.bookkeeper.mledger.util.SafeRun.safeRun;
2222
import com.google.common.collect.Lists;
2323
import com.google.common.collect.Maps;
24-
import io.netty.buffer.ByteBuf;
2524
import java.util.concurrent.ConcurrentMap;
2625
import java.util.concurrent.TimeUnit;
2726
import java.util.concurrent.atomic.AtomicBoolean;
2827
import java.util.concurrent.atomic.AtomicLong;
29-
import org.apache.bookkeeper.client.api.LedgerEntry;
30-
import org.apache.bookkeeper.client.impl.LedgerEntryImpl;
31-
import org.apache.bookkeeper.mledger.Entry;
32-
import org.apache.bookkeeper.mledger.impl.EntryImpl;
3328
import org.apache.bookkeeper.mledger.impl.ManagedLedgerFactoryImpl;
3429
import org.apache.bookkeeper.mledger.impl.ManagedLedgerFactoryMBeanImpl;
3530
import org.apache.bookkeeper.mledger.impl.ManagedLedgerImpl;
36-
import org.apache.bookkeeper.mledger.intercept.ManagedLedgerInterceptor;
3731
import org.slf4j.Logger;
3832
import org.slf4j.LoggerFactory;
3933

@@ -168,29 +162,9 @@ public void clear() {
168162
caches.values().forEach(EntryCache::clear);
169163
}
170164

171-
public static Entry create(long ledgerId, long entryId, ByteBuf data) {
172-
return EntryImpl.create(ledgerId, entryId, data);
173-
}
174-
175-
public static EntryImpl create(LedgerEntry ledgerEntry, ManagedLedgerInterceptor interceptor) {
176-
ManagedLedgerInterceptor.PayloadProcessorHandle processorHandle = null;
177-
if (interceptor != null) {
178-
ByteBuf duplicateBuffer = ledgerEntry.getEntryBuffer().retainedDuplicate();
179-
processorHandle = interceptor
180-
.processPayloadBeforeEntryCache(duplicateBuffer);
181-
if (processorHandle != null) {
182-
ledgerEntry = LedgerEntryImpl.create(ledgerEntry.getLedgerId(), ledgerEntry.getEntryId(),
183-
ledgerEntry.getLength(), processorHandle.getProcessedPayload());
184-
} else {
185-
duplicateBuffer.release();
186-
}
187-
}
188-
EntryImpl returnEntry = EntryImpl.create(ledgerEntry);
189-
if (processorHandle != null) {
190-
processorHandle.release();
191-
ledgerEntry.close();
192-
}
193-
return returnEntry;
165+
@Override
166+
public void close() {
167+
clear();
194168
}
195169

196170
private static final Logger log = LoggerFactory.getLogger(RangeEntryCacheManagerImpl.class);
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,35 @@
1+
/**
2+
* Licensed to the Apache Software Foundation (ASF) under one
3+
* or more contributor license agreements. See the NOTICE file
4+
* distributed with this work for additional information
5+
* regarding copyright ownership. The ASF licenses this file
6+
* to you under the Apache License, Version 2.0 (the
7+
* "License"); you may not use this file except in compliance
8+
* with the License. You may obtain a copy of the License at
9+
*
10+
* http://www.apache.org/licenses/LICENSE-2.0
11+
*
12+
* Unless required by applicable law or agreed to in writing,
13+
* software distributed under the License is distributed on an
14+
* "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
15+
* KIND, either express or implied. See the License for the
16+
* specific language governing permissions and limitations
17+
* under the License.
18+
*/
19+
package org.apache.bookkeeper.mledger.impl.cache;
20+
21+
import io.netty.buffer.ByteBuf;
22+
23+
public interface SharedCacheSegment extends AutoCloseable {
24+
25+
boolean insert(long ledgerId, long entryId, ByteBuf entry);
26+
27+
ByteBuf get(long ledgerId, long entryId);
28+
29+
int getSize();
30+
31+
void clear();
32+
33+
@Override
34+
void close();
35+
}
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,99 @@
1+
/**
2+
* Licensed to the Apache Software Foundation (ASF) under one
3+
* or more contributor license agreements. See the NOTICE file
4+
* distributed with this work for additional information
5+
* regarding copyright ownership. The ASF licenses this file
6+
* to you under the Apache License, Version 2.0 (the
7+
* "License"); you may not use this file except in compliance
8+
* with the License. You may obtain a copy of the License at
9+
*
10+
* http://www.apache.org/licenses/LICENSE-2.0
11+
*
12+
* Unless required by applicable law or agreed to in writing,
13+
* software distributed under the License is distributed on an
14+
* "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
15+
* KIND, either express or implied. See the License for the
16+
* specific language governing permissions and limitations
17+
* under the License.
18+
*/
19+
package org.apache.bookkeeper.mledger.impl.cache;
20+
21+
import io.netty.buffer.ByteBuf;
22+
import java.util.concurrent.atomic.AtomicInteger;
23+
import org.apache.pulsar.common.allocator.PulsarByteBufAllocator;
24+
import org.apache.pulsar.common.util.collections.ConcurrentLongLongPairHashMap;
25+
26+
class SharedCacheSegmentBufferCopy implements AutoCloseable, SharedCacheSegment {
27+
28+
private final ByteBuf cacheBuffer;
29+
private final AtomicInteger currentOffset = new AtomicInteger();
30+
private final ConcurrentLongLongPairHashMap index;
31+
private final int segmentSize;
32+
33+
private static final int ALIGN_64_MASK = ~(64 - 1);
34+
35+
SharedCacheSegmentBufferCopy(int segmentSize) {
36+
this.segmentSize = segmentSize;
37+
this.cacheBuffer = PulsarByteBufAllocator.DEFAULT.buffer(segmentSize, segmentSize);
38+
this.cacheBuffer.writerIndex(segmentSize - 1);
39+
this.index = ConcurrentLongLongPairHashMap.newBuilder()
40+
// We are going to often clear() the map, with the expectation that it's going to get filled again
41+
// immediately after. In these conditions it does not make sense to shrink it each time.
42+
.autoShrink(false)
43+
.concurrencyLevel(Runtime.getRuntime().availableProcessors() * 8)
44+
.build();
45+
}
46+
47+
@Override
48+
public boolean insert(long ledgerId, long entryId, ByteBuf entry) {
49+
int entrySize = entry.readableBytes();
50+
int alignedSize = align64(entrySize);
51+
int offset = currentOffset.getAndAdd(alignedSize);
52+
53+
if (offset + entrySize > segmentSize) {
54+
// The segment is full
55+
return false;
56+
} else {
57+
// Copy entry into read cache segment
58+
cacheBuffer.setBytes(offset, entry, entry.readerIndex(), entry.readableBytes());
59+
long value = offset << 32 | entrySize;
60+
index.put(ledgerId, entryId, value, 0);
61+
return true;
62+
}
63+
}
64+
65+
@Override
66+
public ByteBuf get(long ledgerId, long entryId) {
67+
long value = index.getFirstValue(ledgerId, entryId);
68+
if (value >= 0) {
69+
int offset = (int) (value >> 32);
70+
int entryLen = (int) value;
71+
72+
ByteBuf entry = PulsarByteBufAllocator.DEFAULT.buffer(entryLen, entryLen);
73+
entry.writeBytes(cacheBuffer, offset, entryLen);
74+
return entry;
75+
} else {
76+
return null;
77+
}
78+
}
79+
80+
@Override
81+
public int getSize() {
82+
return currentOffset.get();
83+
}
84+
85+
@Override
86+
public void close() {
87+
cacheBuffer.release();
88+
}
89+
90+
private static int align64(int size) {
91+
return (size + 64 - 1) & ALIGN_64_MASK;
92+
}
93+
94+
@Override
95+
public void clear() {
96+
index.clear();
97+
currentOffset.set(0);
98+
}
99+
}

0 commit comments

Comments
 (0)