forked from apache/pulsar
-
Notifications
You must be signed in to change notification settings - Fork 0
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
[improve][broker] Reduce the pressure from the transaction buffer rea…
…ders and writers in rolling restarts ### Motivation During the rolling restarts, the namespace bundle ownerships will change. Assuming there is a producer created on a single topic, and the ownership was transferred to the new broker. Assuming the namespace bundle has N topics and the namespace is `tenant/ns`, 1. All N topics in the same bundle of that topic will be loaded. 2. For each topic, the managed ledger will be initialized, when the transaction coordinator is enabled, a `TopicTransactionBuffer` will be created. 2.1 A Pulsar producer will be created on `tenant/ns/__transaction_buffer_snapshot` concurrently. 2.2 A Pulsar reader will be created on `tenant/ns/__transaction_buffer_snapshot` concurrently. 3. Once all N readers are created, the owner of the snapshot topic will start dispatching messages to N readers. Each dispatcher will read messages from BookKeeper concurrently and might fail with too many requests error because BK can only have `maxPendingReadRequestsPerThread` pending read requests (default: 10000). We have a `numTransactionReplayThreadPoolSize` config to limit the concurrency of transaction snapshot readers. However, it only limits the read loop. For example, if it's configured with 1, only 1 reader could read messages at the same time. However, N readers will be created concurrently. Each when one of these reader explicitly calls `readNext`, all N dispatchers at brokers side will dispatch messages to N readers. The behaviors above brings much CPU pressure on the owner broker, especially for a small cluster with only two brokers. ### Modifications - Synchronize the reader creation, read loop and the following process on its result. Maintain only one reader for each namespace.
- Loading branch information
1 parent
c50f4af
commit ffcc578
Showing
5 changed files
with
246 additions
and
54 deletions.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
86 changes: 86 additions & 0 deletions
86
...ker/src/main/java/org/apache/pulsar/broker/transaction/buffer/impl/SnapshotTableView.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,86 @@ | ||
/* | ||
* Licensed to the Apache Software Foundation (ASF) under one | ||
* or more contributor license agreements. See the NOTICE file | ||
* distributed with this work for additional information | ||
* regarding copyright ownership. The ASF licenses this file | ||
* to you under the Apache License, Version 2.0 (the | ||
* "License"); you may not use this file except in compliance | ||
* with the License. You may obtain a copy of the License at | ||
* | ||
* http://www.apache.org/licenses/LICENSE-2.0 | ||
* | ||
* Unless required by applicable law or agreed to in writing, | ||
* software distributed under the License is distributed on an | ||
* "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY | ||
* KIND, either express or implied. See the License for the | ||
* specific language governing permissions and limitations | ||
* under the License. | ||
*/ | ||
package org.apache.pulsar.broker.transaction.buffer.impl; | ||
|
||
import static org.apache.pulsar.broker.systopic.SystemTopicClient.Reader; | ||
import java.util.Map; | ||
import java.util.concurrent.CompletableFuture; | ||
import java.util.concurrent.ConcurrentHashMap; | ||
import java.util.concurrent.ExecutionException; | ||
import java.util.concurrent.ScheduledExecutorService; | ||
import java.util.concurrent.TimeUnit; | ||
import lombok.extern.slf4j.Slf4j; | ||
import org.apache.pulsar.broker.service.SystemTopicTxnBufferSnapshotService; | ||
import org.apache.pulsar.broker.transaction.buffer.metadata.TransactionBufferSnapshot; | ||
import org.apache.pulsar.common.naming.NamespaceName; | ||
import org.apache.pulsar.common.naming.TopicName; | ||
import org.apache.pulsar.utils.SimpleCache; | ||
|
||
/** | ||
* Compared with the more generic {@link org.apache.pulsar.client.api.TableView}, this table view | ||
* - Provides just a single public method that reads the latest value synchronously. | ||
* - Maintains multiple long-lived readers that will be expired after some time (1 minute by default). | ||
*/ | ||
@Slf4j | ||
public class SnapshotTableView { | ||
|
||
// Remove the cached reader and snapshots if there is no refresh request in 1 minute | ||
private static final long CACHE_EXPIRE_TIMEOUT_MS = 60 * 1000L; | ||
private final Map<String, TransactionBufferSnapshot> snapshots = new ConcurrentHashMap<>(); | ||
private final SystemTopicTxnBufferSnapshotService<TransactionBufferSnapshot> snapshotService; | ||
private final long blockTimeoutMs; | ||
private final SimpleCache<NamespaceName, Reader<TransactionBufferSnapshot>> readers; | ||
|
||
public SnapshotTableView(SystemTopicTxnBufferSnapshotService<TransactionBufferSnapshot> snapshotService, | ||
ScheduledExecutorService executor, long blockTimeoutMs) { | ||
this.snapshotService = snapshotService; | ||
this.blockTimeoutMs = blockTimeoutMs; | ||
this.readers = new SimpleCache<>(executor, CACHE_EXPIRE_TIMEOUT_MS); | ||
} | ||
|
||
public TransactionBufferSnapshot readLatest(String topic) throws Exception { | ||
final var topicName = TopicName.get(topic); | ||
final var namespace = topicName.getNamespaceObject(); | ||
final var reader = readers.get(namespace, () -> { | ||
try { | ||
return wait(snapshotService.createReader(topicName), "create reader"); | ||
} catch (Exception e) { | ||
throw new RuntimeException(e); | ||
} | ||
}, __ -> __.closeAsync().exceptionally(e -> { | ||
log.warn("Failed to close reader {}", e.getMessage()); | ||
return null; | ||
})); | ||
while (wait(reader.hasMoreEventsAsync(), "has more events")) { | ||
final var msg = wait(reader.readNextAsync(), "read message"); | ||
if (msg.getKey() != null) { | ||
snapshots.put(msg.getKey(), msg.getValue()); | ||
} | ||
} | ||
return snapshots.get(topic); | ||
} | ||
|
||
private <T> T wait(CompletableFuture<T> future, String msg) throws Exception { | ||
try { | ||
return future.get(blockTimeoutMs, TimeUnit.MILLISECONDS); | ||
} catch (ExecutionException e) { | ||
throw new ExecutionException("Failed to " + msg, e.getCause()); | ||
} | ||
} | ||
} |
62 changes: 62 additions & 0 deletions
62
pulsar-broker/src/main/java/org/apache/pulsar/utils/SimpleCache.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,62 @@ | ||
/* | ||
* Licensed to the Apache Software Foundation (ASF) under one | ||
* or more contributor license agreements. See the NOTICE file | ||
* distributed with this work for additional information | ||
* regarding copyright ownership. The ASF licenses this file | ||
* to you under the Apache License, Version 2.0 (the | ||
* "License"); you may not use this file except in compliance | ||
* with the License. You may obtain a copy of the License at | ||
* | ||
* http://www.apache.org/licenses/LICENSE-2.0 | ||
* | ||
* Unless required by applicable law or agreed to in writing, | ||
* software distributed under the License is distributed on an | ||
* "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY | ||
* KIND, either express or implied. See the License for the | ||
* specific language governing permissions and limitations | ||
* under the License. | ||
*/ | ||
package org.apache.pulsar.utils; | ||
|
||
import java.util.HashMap; | ||
import java.util.Map; | ||
import java.util.concurrent.ScheduledExecutorService; | ||
import java.util.concurrent.ScheduledFuture; | ||
import java.util.concurrent.TimeUnit; | ||
import java.util.function.Consumer; | ||
import java.util.function.Supplier; | ||
import lombok.RequiredArgsConstructor; | ||
|
||
@RequiredArgsConstructor | ||
public class SimpleCache<K, V> { | ||
|
||
private final Map<K, V> cache = new HashMap<>(); | ||
private final Map<K, ScheduledFuture<?>> futures = new HashMap<>(); | ||
private final ScheduledExecutorService executor; | ||
private final long timeoutMs; | ||
|
||
public synchronized V get(final K key, final Supplier<V> valueSupplier, final Consumer<V> expireCallback) { | ||
final V value; | ||
V existingValue = cache.get(key); | ||
if (existingValue != null) { | ||
value = existingValue; | ||
} else { | ||
value = valueSupplier.get(); | ||
cache.put(key, value); | ||
} | ||
final var future = futures.remove(key); | ||
if (future != null) { | ||
future.cancel(true); | ||
} | ||
futures.put(key, executor.schedule(() -> { | ||
synchronized (SimpleCache.this) { | ||
futures.remove(key); | ||
final var removedValue = cache.remove(key); | ||
if (removedValue != null) { | ||
expireCallback.accept(removedValue); | ||
} | ||
} | ||
}, timeoutMs, TimeUnit.MILLISECONDS)); | ||
return value; | ||
} | ||
} |
74 changes: 74 additions & 0 deletions
74
pulsar-broker/src/test/java/org/apache/pulsar/utils/SimpleCacheTest.java
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,74 @@ | ||
/* | ||
* Licensed to the Apache Software Foundation (ASF) under one | ||
* or more contributor license agreements. See the NOTICE file | ||
* distributed with this work for additional information | ||
* regarding copyright ownership. The ASF licenses this file | ||
* to you under the Apache License, Version 2.0 (the | ||
* "License"); you may not use this file except in compliance | ||
* with the License. You may obtain a copy of the License at | ||
* | ||
* http://www.apache.org/licenses/LICENSE-2.0 | ||
* | ||
* Unless required by applicable law or agreed to in writing, | ||
* software distributed under the License is distributed on an | ||
* "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY | ||
* KIND, either express or implied. See the License for the | ||
* specific language governing permissions and limitations | ||
* under the License. | ||
*/ | ||
package org.apache.pulsar.utils; | ||
|
||
import java.util.Collections; | ||
import java.util.concurrent.CopyOnWriteArrayList; | ||
import java.util.concurrent.CountDownLatch; | ||
import java.util.concurrent.Executors; | ||
import java.util.concurrent.ScheduledExecutorService; | ||
import org.testng.Assert; | ||
import org.testng.annotations.AfterClass; | ||
import org.testng.annotations.Test; | ||
|
||
public class SimpleCacheTest { | ||
|
||
private final ScheduledExecutorService executor = Executors.newScheduledThreadPool(1); | ||
|
||
@AfterClass | ||
public void shutdown() { | ||
executor.shutdown(); | ||
} | ||
|
||
@Test | ||
public void testConcurrentUpdate() throws Exception { | ||
final var cache = new SimpleCache<Integer, Integer>(executor, 10000L); | ||
final var pool = Executors.newFixedThreadPool(2); | ||
final var latch = new CountDownLatch(2); | ||
for (int i = 0; i < 2; i++) { | ||
final var value = i + 100; | ||
pool.execute(() -> { | ||
try { | ||
Thread.sleep(100); | ||
} catch (InterruptedException ignored) { | ||
} | ||
cache.get(0, () -> value, __ -> {}); | ||
latch.countDown(); | ||
}); | ||
} | ||
latch.await(); | ||
final var value = cache.get(0, () -> -1, __ -> {}); | ||
Assert.assertTrue(value == 100 || value == 101); | ||
pool.shutdown(); | ||
} | ||
|
||
@Test | ||
public void testExpire() throws InterruptedException { | ||
final var cache = new SimpleCache<Integer, Integer>(executor, 500L); | ||
final var expiredValues = new CopyOnWriteArrayList<Integer>(); | ||
cache.get(0, () -> 100, expiredValues::add); | ||
for (int i = 0; i < 100; i++) { | ||
cache.get(1, () -> 101, expiredValues::add); | ||
Thread.sleep(10); | ||
} | ||
Assert.assertEquals(cache.get(0, () -> -1, __ -> {}), -1); // the value is expired | ||
Assert.assertEquals(cache.get(1, () -> -1, __ -> {}), 101); | ||
Assert.assertEquals(expiredValues, Collections.singletonList(100)); | ||
} | ||
} |