Skip to content

Commit ee7fbbb

Browse files
committed
Implement leader rate limiting for file restore (#37677)
This is related to #35975. This commit implements rate limiting on the leader side using the CombinedRateLimiter.
1 parent 5dfe193 commit ee7fbbb

File tree

2 files changed

+38
-5
lines changed

2 files changed

+38
-5
lines changed

x-pack/plugin/ccr/src/main/java/org/elasticsearch/xpack/ccr/repository/CcrRestoreSourceService.java

Lines changed: 17 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -16,8 +16,10 @@
1616
import org.elasticsearch.common.bytes.BytesReference;
1717
import org.elasticsearch.common.component.AbstractLifecycleComponent;
1818
import org.elasticsearch.common.lease.Releasable;
19+
import org.elasticsearch.common.metrics.CounterMetric;
1920
import org.elasticsearch.common.settings.Settings;
2021
import org.elasticsearch.common.unit.TimeValue;
22+
import org.elasticsearch.common.util.CombinedRateLimiter;
2123
import org.elasticsearch.common.util.concurrent.AbstractRefCounted;
2224
import org.elasticsearch.common.util.concurrent.ConcurrentCollections;
2325
import org.elasticsearch.common.util.concurrent.KeyedLock;
@@ -42,6 +44,7 @@
4244
import java.util.concurrent.ConcurrentHashMap;
4345
import java.util.concurrent.CopyOnWriteArrayList;
4446
import java.util.function.Consumer;
47+
import java.util.function.LongConsumer;
4548

4649
public class CcrRestoreSourceService extends AbstractLifecycleComponent implements IndexEventListener {
4750

@@ -52,6 +55,7 @@ public class CcrRestoreSourceService extends AbstractLifecycleComponent implemen
5255
private final CopyOnWriteArrayList<Consumer<String>> closeSessionListeners = new CopyOnWriteArrayList<>();
5356
private final ThreadPool threadPool;
5457
private final CcrSettings ccrSettings;
58+
private final CounterMetric throttleTime = new CounterMetric();
5559

5660
public CcrRestoreSourceService(ThreadPool threadPool, CcrSettings ccrSettings) {
5761
this.threadPool = threadPool;
@@ -136,7 +140,7 @@ public synchronized SessionReader getSessionReader(String sessionUUID) {
136140
throw new IllegalArgumentException("session [" + sessionUUID + "] not found");
137141
}
138142
restore.idle = false;
139-
return new SessionReader(restore);
143+
return new SessionReader(restore, ccrSettings, throttleTime::inc);
140144
}
141145

142146
private void internalCloseSession(String sessionUUID, boolean throwIfSessionMissing) {
@@ -182,6 +186,10 @@ private void maybeTimeout(String sessionUUID) {
182186
}
183187
}
184188

189+
public long getThrottleTime() {
190+
return this.throttleTime.count();
191+
}
192+
185193
private static class RestoreSession extends AbstractRefCounted {
186194

187195
private final String sessionUUID;
@@ -254,9 +262,13 @@ protected void closeInternal() {
254262
public static class SessionReader implements Closeable {
255263

256264
private final RestoreSession restoreSession;
265+
private final CcrSettings ccrSettings;
266+
private final LongConsumer throttleListener;
257267

258-
private SessionReader(RestoreSession restoreSession) {
268+
private SessionReader(RestoreSession restoreSession, CcrSettings ccrSettings, LongConsumer throttleListener) {
259269
this.restoreSession = restoreSession;
270+
this.ccrSettings = ccrSettings;
271+
this.throttleListener = throttleListener;
260272
restoreSession.incRef();
261273
}
262274

@@ -270,6 +282,9 @@ private SessionReader(RestoreSession restoreSession) {
270282
* @throws IOException if the read fails
271283
*/
272284
public long readFileBytes(String fileName, BytesReference reference) throws IOException {
285+
CombinedRateLimiter rateLimiter = ccrSettings.getRateLimiter();
286+
long throttleTime = rateLimiter.maybePause(reference.length());
287+
throttleListener.accept(throttleTime);
273288
return restoreSession.readFileBytes(fileName, reference);
274289
}
275290

x-pack/plugin/ccr/src/test/java/org/elasticsearch/xpack/ccr/CcrRepositoryIT.java

Lines changed: 21 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -239,9 +239,15 @@ public void testDocsAreRecovered() throws Exception {
239239
}
240240

241241
public void testRateLimitingIsEmployed() throws Exception {
242+
boolean followerRateLimiting = randomBoolean();
243+
242244
ClusterUpdateSettingsRequest settingsRequest = new ClusterUpdateSettingsRequest();
243245
settingsRequest.persistentSettings(Settings.builder().put(CcrSettings.RECOVERY_MAX_BYTES_PER_SECOND.getKey(), "10K"));
244-
assertAcked(followerClient().admin().cluster().updateSettings(settingsRequest).actionGet());
246+
if (followerRateLimiting) {
247+
assertAcked(followerClient().admin().cluster().updateSettings(settingsRequest).actionGet());
248+
} else {
249+
assertAcked(leaderClient().admin().cluster().updateSettings(settingsRequest).actionGet());
250+
}
245251

246252
String leaderClusterRepoName = CcrRepository.NAME_PREFIX + "leader_cluster";
247253
String leaderIndex = "index1";
@@ -257,11 +263,15 @@ public void testRateLimitingIsEmployed() throws Exception {
257263
final ClusterService clusterService = getFollowerCluster().getCurrentMasterNodeInstance(ClusterService.class);
258264

259265
List<CcrRepository> repositories = new ArrayList<>();
266+
List<CcrRestoreSourceService> restoreSources = new ArrayList<>();
260267

261268
for (RepositoriesService repositoriesService : getFollowerCluster().getDataOrMasterNodeInstances(RepositoriesService.class)) {
262269
Repository repository = repositoriesService.repository(leaderClusterRepoName);
263270
repositories.add((CcrRepository) repository);
264271
}
272+
for (CcrRestoreSourceService restoreSource : getLeaderCluster().getDataOrMasterNodeInstances(CcrRestoreSourceService.class)) {
273+
restoreSources.add(restoreSource);
274+
}
265275

266276
logger.info("--> indexing some data");
267277
for (int i = 0; i < 100; i++) {
@@ -284,12 +294,20 @@ public void testRateLimitingIsEmployed() throws Exception {
284294
restoreService.restoreSnapshot(restoreRequest, waitForRestore(clusterService, future));
285295
future.actionGet();
286296

287-
assertTrue(repositories.stream().anyMatch(cr -> cr.getRestoreThrottleTimeInNanos() > 0));
297+
if (followerRateLimiting) {
298+
assertTrue(repositories.stream().anyMatch(cr -> cr.getRestoreThrottleTimeInNanos() > 0));
299+
} else {
300+
assertTrue(restoreSources.stream().anyMatch(cr -> cr.getThrottleTime() > 0));
301+
}
288302

289303
settingsRequest = new ClusterUpdateSettingsRequest();
290304
ByteSizeValue defaultValue = CcrSettings.RECOVERY_MAX_BYTES_PER_SECOND.getDefault(Settings.EMPTY);
291305
settingsRequest.persistentSettings(Settings.builder().put(CcrSettings.RECOVERY_MAX_BYTES_PER_SECOND.getKey(), defaultValue));
292-
assertAcked(followerClient().admin().cluster().updateSettings(settingsRequest).actionGet());
306+
if (followerRateLimiting) {
307+
assertAcked(followerClient().admin().cluster().updateSettings(settingsRequest).actionGet());
308+
} else {
309+
assertAcked(leaderClient().admin().cluster().updateSettings(settingsRequest).actionGet());
310+
}
293311
}
294312

295313
@AwaitsFix(bugUrl = "https://github.com/elastic/elasticsearch/issues/37887")

0 commit comments

Comments
 (0)