Skip to content

Conversation

@lhotari
Copy link
Member

@lhotari lhotari commented May 29, 2025

Motivation

This PR fixes fundamental inefficiencies and correctness issues in the current Pulsar broker entry cache eviction algorithm. The current implementation has flawed size-based eviction that doesn't remove the oldest entries and incorrect timestamp-based eviction with high CPU overhead. These fixes ensure that size-based eviction properly removes the oldest entries and timestamp-based eviction works correctly. Additionally, this PR serves as a foundation for future improvements to efficiently handle catch-up reads and Key_Shared subscription scenarios.

Mailing list discussion about this PR: https://lists.apache.org/thread/ddzzc17b0c218ozq9tx0r3rx5sgljfb0

UPDATE: This change is covered in "PIP-430: Pulsar Broker cache improvements: refactoring eviction and adding a new cache strategy based on expected read count", #24444

Problems with the Current Broker Entry Cache Implementation

  1. Size-Based Eviction doesn't remove oldest entries: The existing EntryCacheDefaultEvictionPolicy uses an algorithm for keeping the cache size under the limit but cannot guarantee removal of the oldest entries from the cache. The algorithm:

    • Sorts caches by size in descending order
    • Selects caches representing a percentage of total size (PercentOfSizeToConsiderForEviction, default 0.5)
    • Attempts to evict proportional amounts from each selected cache
    • This approach doesn't ensure that the oldest entries are removed, leading to inefficient cache utilization
  2. Inefficient and Incorrect Timestamp-Based Eviction: The current timestamp eviction has both performance and correctness issues:

    • Performance problems:
      • Iterates through ALL cache instances (there's an instance for each persistent topic / ManagedLedgerImpl)
      • Causes remarkable CPU and memory pressure due to constant frequent iterations when there's a high number of topics running with high throughput.
      • Runs every 10 milliseconds by default (managedLedgerCacheEvictionIntervalMs=10) - 100 times per second!
    • Correctness problems:
      • Assumes entries are ordered by both position and timestamp, which breaks when:
        • Cache is used for catch-up reads (backlogged cursors)
        • Individual entries are cached out of order
        • Multiple read patterns (tailing + catch-up) access the same cache simultaneously
  3. Limited Cache Scope: The original RangeCache was designed for tailing reads. Later changes added support for backlogged cursors, but the eviction algorithms weren't updated to handle mixed read patterns effectively.

  4. Unnecessary Complexity: Generic type parameters in RangeCache add complexity without providing value, as the cache is only used for entry storage.

Modifications

1. Centralized Removal Queue (RangeCacheRemovalQueue)

  • Single-threaded eviction: All cache evictions are handled by one thread to avoid contention
  • Insertion-order tracking: Uses MpscUnboundedArrayQueue to maintain entry insertion order
  • Accurate timestamp eviction: Entries are processed in insertion order, making timestamp-based eviction reliable
  • Efficient size-based eviction: Oldest entries are removed first when freeing cache space

2. Simplified Cache Implementation

  • Removed generics: Dropped unnecessary type parameters from RangeCache to reduce complexity
  • Unified eviction handling: All cache instances use the same central removal queue
  • Improved consistency: Race conditions in eviction are minimized through centralized processing

3. Foundation for Future Improvements

The existing broker cache has limitations:

  • Unnecessary BookKeeper reads during catch-up read scenarios
    • Causes increased network costs and resource usage on BookKeeper nodes
    • Cascading performance issues under high fan-out catch-up reads
    • Current backlogged cursors caching solution has multiple gaps
  • Poor cache hit rates for Key_Shared subscriptions with slow consumers since entries get put into the replay queue and once the consumer has sent permits, these entries are read from BookKeeper (unless cacheEvictionByMarkDeletedPosition=true)

This refactoring prepares the cache system for:

  • Enhanced catch-up read optimization
  • Efficient replay queue caching for Key_Shared subscriptions

Algorithm Comparison

Before (EntryCacheDefaultEvictionPolicy)

Size Based Eviction

  1. Sort all caches by size (largest first)
  2. Select caches until reaching PercentOfSizeToConsiderForEviction (0.5)
  3. For each selected cache:
    • Calculate proportional eviction amount
    • Remove entries (no guarantee of age-based removal)
  4. Problem: May remove newer entries while keeping older ones

@Override
public void doEviction(List<EntryCache> caches, long sizeToFree) {
checkArgument(sizeToFree > 0);
checkArgument(!caches.isEmpty());
caches.sort(reverseOrder());
long totalSize = 0;
for (EntryCache cache : caches) {
totalSize += cache.getSize();
}
// This algorithm apply the eviction only the group of caches whose combined size reaches the
// PercentOfSizeToConsiderForEviction
List<EntryCache> cachesToEvict = new ArrayList();
long cachesToEvictTotalSize = 0;
long sizeToConsiderForEviction = (long) (totalSize * PercentOfSizeToConsiderForEviction);
log.debug("Need to gather at least {} from caches", sizeToConsiderForEviction);
int cacheIdx = 0;
while (cachesToEvictTotalSize < sizeToConsiderForEviction) {
// This condition should always be true, considering that we cannot free more size that what we have in
// cache
checkArgument(cacheIdx < caches.size());
EntryCache entryCache = caches.get(cacheIdx++);
cachesToEvictTotalSize += entryCache.getSize();
cachesToEvict.add(entryCache);
log.debug("Added cache {} with size {}", entryCache.getName(), entryCache.getSize());
}
int evictedEntries = 0;
long evictedSize = 0;
for (EntryCache entryCache : cachesToEvict) {
// To each entryCache chosen to for eviction, we'll ask to evict a proportional amount of data
long singleCacheSizeToFree = (long) (sizeToFree * (entryCache.getSize() / (double) cachesToEvictTotalSize));
if (singleCacheSizeToFree == 0) {
// If the size of this cache went to 0, it probably means that its entries has been removed from the
// cache since the time we've computed the ranking
continue;
}
Pair<Integer, Long> evicted = entryCache.evictEntries(singleCacheSizeToFree);
evictedEntries += evicted.getLeft();
evictedSize += evicted.getRight();
}
log.info("Completed cache eviction. Removed {} entries from {} caches. ({} Mb)", evictedEntries,
cachesToEvict.size(), evictedSize / RangeEntryCacheManagerImpl.MB);
}

Timestamp eviction

  1. Iterate all caches
  2. For each cache:
    • Start from the first position in the cache
    • Remove entries from the cache until the cache is empty or there's a valid entry that hasn't yet been expired
  3. Problem: Iterating all caches and entries cause a lot of unnecessary CPU and memory pressure due to iterations. By default, this is performed every 10 milliseconds, 100 times per second. (managedLedgerCacheEvictionIntervalMs=10)

private synchronized void doCacheEviction() {
long maxTimestamp = System.nanoTime() - cacheEvictionTimeThresholdNanos;
ledgers.values().forEach(mlfuture -> {
if (mlfuture.isDone() && !mlfuture.isCompletedExceptionally()) {
ManagedLedgerImpl ml = mlfuture.getNow(null);
if (ml != null) {
ml.doCacheEviction(maxTimestamp);
}
}
});
}

After (RangeCacheRemovalQueue)

  1. All entries added to insertion-order queue when cached
  2. For timestamp eviction:
    • Process queue from oldest to newest
    • Remove entries older than threshold
    • Stop when hitting newer entry (leverages insertion order)
  3. For size eviction:
    • Process queue from oldest to newest
    • Remove entries until target size freed
    • Guarantees oldest entries are removed first

https://github.com/apache/pulsar/blob/b72bc4ff3aa5c9c45d9233d2d000429b3cf0ce1a/managed-ledger/src/main/java/org/apache/bookkeeper/mledger/impl/cache/RangeCacheRemovalQueue.java

Note: There's a single shared removal queue for all ManagedLedgerImpl instances instead of having to do the check in multiple instances.

Verifying this change

This change is already covered by existing tests:

  • All existing cache-related tests continue to pass
  • RangeCacheTest validates the new removal queue functionality
  • EntryCacheManagerTest verifies eviction behavior remains correct
  • Integration tests ensure no regression in cache performance

Documentation

  • doc
  • doc-required
  • doc-not-needed
  • doc-complete

@lhotari lhotari added this to the 4.1.0 milestone May 29, 2025
@lhotari lhotari self-assigned this May 29, 2025
@lhotari lhotari requested review from dao-jun, merlimat and nodece May 29, 2025 18:32
@github-actions github-actions bot added the doc-not-needed Your PR changes do not impact docs label May 29, 2025
@codecov-commenter
Copy link

codecov-commenter commented May 29, 2025

Codecov Report

Attention: Patch coverage is 87.80488% with 45 lines in your changes missing coverage. Please review.

Project coverage is 74.36%. Comparing base (bbc6224) to head (b62ccdd).
Report is 1223 commits behind head on master.

Files with missing lines Patch % Lines
...ache/bookkeeper/mledger/impl/cache/RangeCache.java 79.56% 9 Missing and 10 partials ⚠️
...kkeeper/mledger/impl/ManagedLedgerFactoryImpl.java 73.52% 8 Missing and 1 partial ⚠️
...l/cache/RangeEntryCacheManagerEvictionHandler.java 76.47% 2 Missing and 2 partials ⚠️
...che/bookkeeper/mledger/impl/ManagedLedgerImpl.java 85.71% 0 Missing and 3 partials ⚠️
...per/mledger/impl/cache/RangeCacheEntryWrapper.java 93.75% 1 Missing and 2 partials ⚠️
...mledger/impl/cache/RangeEntryCacheManagerImpl.java 92.50% 0 Missing and 3 partials ⚠️
.../org/apache/bookkeeper/mledger/impl/EntryImpl.java 93.75% 1 Missing and 1 partial ⚠️
...main/java/org/apache/bookkeeper/mledger/Entry.java 0.00% 1 Missing ⚠️
...keeper/mledger/impl/cache/PendingReadsManager.java 80.00% 1 Missing ⚠️
Additional details and impacted files

Impacted file tree graph

@@             Coverage Diff              @@
##             master   #24363      +/-   ##
============================================
+ Coverage     73.57%   74.36%   +0.78%     
- Complexity    32624    32651      +27     
============================================
  Files          1877     1878       +1     
  Lines        139502   146301    +6799     
  Branches      15299    16772    +1473     
============================================
+ Hits         102638   108796    +6158     
+ Misses        28908    28883      -25     
- Partials       7956     8622     +666     
Flag Coverage Δ
inttests 26.69% <63.14%> (+2.10%) ⬆️
systests 23.29% <60.16%> (-1.03%) ⬇️
unittests 73.86% <87.80%> (+1.02%) ⬆️

Flags with carried forward coverage won't be shown. Click here to find out more.

Files with missing lines Coverage Δ
...org/apache/bookkeeper/mledger/PositionFactory.java 100.00% <100.00%> (ø)
...kkeeper/mledger/impl/cache/EntryCacheDisabled.java 75.55% <100.00%> (+4.72%) ⬆️
.../mledger/impl/cache/RangeCacheRemovalCounters.java 100.00% <100.00%> (ø)
...per/mledger/impl/cache/RangeCacheRemovalQueue.java 100.00% <100.00%> (ø)
...keeper/mledger/impl/cache/RangeEntryCacheImpl.java 63.30% <100.00%> (+4.55%) ⬆️
...oker/service/nonpersistent/NonPersistentTopic.java 72.89% <ø> (+3.42%) ⬆️
...main/java/org/apache/bookkeeper/mledger/Entry.java 0.00% <0.00%> (ø)
...keeper/mledger/impl/cache/PendingReadsManager.java 87.77% <80.00%> (+1.10%) ⬆️
.../org/apache/bookkeeper/mledger/impl/EntryImpl.java 86.91% <93.75%> (+8.43%) ⬆️
...che/bookkeeper/mledger/impl/ManagedLedgerImpl.java 81.27% <85.71%> (+0.61%) ⬆️
... and 5 more

... and 1105 files with indirect coverage changes

🚀 New features to boost your workflow:
  • ❄️ Test Analytics: Detect flaky tests, report on failures, and find test suite problems.
  • 📦 JS Bundle Analysis: Save yourself from yourself by tracking and limiting bundle sizes in JS merges.

Copy link
Contributor

@merlimat merlimat left a comment

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

My only question is about the eviction from the stash queue: since the insertion might not be in order, when there is memory pressure, how is the eviction going to happen?

Or, is the size of this stash already controlled in size such that it would only require time-based expiration?

@lhotari
Copy link
Member Author

lhotari commented Jul 23, 2025

The merge conflicts will be resolved by first merging #24552 to master branch.

@lhotari
Copy link
Member Author

lhotari commented Jul 23, 2025

My only question is about the eviction from the stash queue: since the insertion might not be in order, when there is memory pressure, how is the eviction going to happen?
Or, is the size of this stash already controlled in size such that it would only require time-based expiration?

For the implementation in this PR all entries are added in order to the removal queue and the timestamp is set when the entry gets added to the cache and the removal queue. Therefore, the timestamps are in order.
When evicting to reduce the cache size under the watermark to handle "memory pressure", oldest entries are evicted from the queue. Time based expiration is done by evicting from the head of the queue until there's a newer timestamp on the entry.

The eviction will become more complicated later. In PIP-430, there's a need to put entries aside and keep them in the cache for a longer period of time. Currently that is addressed in PIP-430 by having 2 separate settings for expiration and that will simplify the solution. The maximum TTL for cache entries will continue to be in order in the removal queue, but another datastructure is needed when size based eviction would remove the entry, but it's prioritized to be skipped and kept until the entry expires. Handling that challenge is out-of-scope for this PR. After this current PR has been merged, it will be possible to build upon this and expand the solution further to implement PIP-430 (I have a WIP PR in my own fork, but it's not matching the PIP high-level design).

One more detail about eviction handling that is present in this PR. The central queue is used for eviction by size (to keep all cached items under the limit) or by TTL. When entries are evicted directly from the RangeCache (for a complete ledger or for up to the markdelete position), the entry wrapper held in the range cache will be cleared and the same instance will remain in the queue until it gets processed. Empty entry wrappers will be held in the queue at most for the TTL in worst case. The entry wrapper gets recycled when it gets processed from the RangeCacheRemovalQueue.

I hope this answer covers your question @merlimat

lhotari added 2 commits July 24, 2025 01:19
… and replace with ReferenceCountedEntry

- ReferenceCountedEntry is already implemented by EntryImpl
- Separate interface to avoid changing and breaking existing Entry interface
@lhotari
Copy link
Member Author

lhotari commented Jul 28, 2025

PIP-430 vote has passed. I'll merge this PR to master.

@lhotari lhotari merged commit 8ce67b9 into apache:master Jul 28, 2025
96 of 98 checks passed
poorbarcode pushed a commit to poorbarcode/pulsar that referenced this pull request Aug 14, 2025
KannarFr pushed a commit to CleverCloud/pulsar that referenced this pull request Sep 22, 2025
walkinggo pushed a commit to walkinggo/pulsar that referenced this pull request Oct 8, 2025
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment

Labels

doc-not-needed Your PR changes do not impact docs ready-to-test

Projects

None yet

Development

Successfully merging this pull request may close these issues.

4 participants