Skip to content

Commit b842ea8

Browse files
Some Cleanup in o.e.i.shard (#44097)
* Some Cleanup in o.e.i.shard * Extract one duplicated method * Cleanup obviously unused code
1 parent 0d0485a commit b842ea8

14 files changed

+67
-105
lines changed

server/src/main/java/org/elasticsearch/index/cache/bitset/BitsetFilterCache.java

Lines changed: 15 additions & 13 deletions
Original file line numberDiff line numberDiff line change
@@ -91,6 +91,19 @@ public BitsetFilterCache(IndexSettings indexSettings, Listener listener) {
9191
this.listener = listener;
9292
}
9393

94+
public static BitSet bitsetFromQuery(Query query, LeafReaderContext context) throws IOException {
95+
final IndexReaderContext topLevelContext = ReaderUtil.getTopLevelContext(context);
96+
final IndexSearcher searcher = new IndexSearcher(topLevelContext);
97+
searcher.setQueryCache(null);
98+
final Weight weight = searcher.createWeight(searcher.rewrite(query), ScoreMode.COMPLETE_NO_SCORES, 1f);
99+
Scorer s = weight.scorer(context);
100+
if (s == null) {
101+
return null;
102+
} else {
103+
return BitSet.of(s.iterator(), context.reader().maxDoc());
104+
}
105+
}
106+
94107
public IndexWarmer.Listener createListener(ThreadPool threadPool) {
95108
return new BitSetProducerWarmer(threadPool);
96109
}
@@ -115,7 +128,7 @@ public void clear(String reason) {
115128
loadedFilters.invalidateAll();
116129
}
117130

118-
private BitSet getAndLoadIfNotPresent(final Query query, final LeafReaderContext context) throws IOException, ExecutionException {
131+
private BitSet getAndLoadIfNotPresent(final Query query, final LeafReaderContext context) throws ExecutionException {
119132
final IndexReader.CacheHelper cacheHelper = context.reader().getCoreCacheHelper();
120133
if (cacheHelper == null) {
121134
throw new IllegalArgumentException("Reader " + context.reader() + " does not support caching");
@@ -133,18 +146,7 @@ private BitSet getAndLoadIfNotPresent(final Query query, final LeafReaderContext
133146
});
134147

135148
return filterToFbs.computeIfAbsent(query, key -> {
136-
final IndexReaderContext topLevelContext = ReaderUtil.getTopLevelContext(context);
137-
final IndexSearcher searcher = new IndexSearcher(topLevelContext);
138-
searcher.setQueryCache(null);
139-
final Weight weight = searcher.createWeight(searcher.rewrite(query), ScoreMode.COMPLETE_NO_SCORES, 1f);
140-
Scorer s = weight.scorer(context);
141-
final BitSet bitSet;
142-
if (s == null) {
143-
bitSet = null;
144-
} else {
145-
bitSet = BitSet.of(s.iterator(), context.reader().maxDoc());
146-
}
147-
149+
final BitSet bitSet = bitsetFromQuery(query, context);
148150
Value value = new Value(bitSet, shardId);
149151
listener.onCache(shardId, value.bitset);
150152
return value;

server/src/main/java/org/elasticsearch/index/shard/AbstractIndexShardComponent.java

Lines changed: 0 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -20,22 +20,19 @@
2020
package org.elasticsearch.index.shard;
2121

2222
import org.apache.logging.log4j.Logger;
23-
import org.elasticsearch.common.logging.DeprecationLogger;
2423
import org.elasticsearch.common.logging.Loggers;
2524
import org.elasticsearch.index.IndexSettings;
2625

2726
public abstract class AbstractIndexShardComponent implements IndexShardComponent {
2827

2928
protected final Logger logger;
30-
protected final DeprecationLogger deprecationLogger;
3129
protected final ShardId shardId;
3230
protected final IndexSettings indexSettings;
3331

3432
protected AbstractIndexShardComponent(ShardId shardId, IndexSettings indexSettings) {
3533
this.shardId = shardId;
3634
this.indexSettings = indexSettings;
3735
this.logger = Loggers.getLogger(getClass(), shardId);
38-
this.deprecationLogger = new DeprecationLogger(logger);
3936
}
4037

4138
@Override
@@ -47,8 +44,4 @@ public ShardId shardId() {
4744
public IndexSettings indexSettings() {
4845
return indexSettings;
4946
}
50-
51-
public String nodeName() {
52-
return indexSettings.getNodeName();
53-
}
5447
}

server/src/main/java/org/elasticsearch/index/shard/ElasticsearchMergePolicy.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -46,7 +46,7 @@
4646
*/
4747
public final class ElasticsearchMergePolicy extends FilterMergePolicy {
4848

49-
private static Logger logger = LogManager.getLogger(ElasticsearchMergePolicy.class);
49+
private static final Logger logger = LogManager.getLogger(ElasticsearchMergePolicy.class);
5050

5151
// True if the next merge request should do segment upgrades:
5252
private volatile boolean upgradeInProgress;

server/src/main/java/org/elasticsearch/index/shard/IndexShard.java

Lines changed: 20 additions & 29 deletions
Original file line numberDiff line numberDiff line change
@@ -40,7 +40,6 @@
4040
import org.elasticsearch.Assertions;
4141
import org.elasticsearch.ElasticsearchException;
4242
import org.elasticsearch.ExceptionsHelper;
43-
import org.elasticsearch.Version;
4443
import org.elasticsearch.action.ActionListener;
4544
import org.elasticsearch.action.admin.indices.flush.FlushRequest;
4645
import org.elasticsearch.action.admin.indices.forcemerge.ForceMergeRequest;
@@ -326,17 +325,15 @@ public IndexShard(
326325
this.pendingPrimaryTerm = primaryTerm;
327326
this.globalCheckpointListeners =
328327
new GlobalCheckpointListeners(shardId, threadPool.executor(ThreadPool.Names.LISTENER), threadPool.scheduler(), logger);
329-
final ReplicationTracker replicationTracker =
330-
new ReplicationTracker(
331-
shardId,
332-
aId,
333-
indexSettings,
334-
primaryTerm,
335-
UNASSIGNED_SEQ_NO,
336-
globalCheckpointListeners::globalCheckpointUpdated,
337-
threadPool::absoluteTimeInMillis,
338-
(retentionLeases, listener) -> retentionLeaseSyncer.sync(shardId, retentionLeases, listener));
339-
this.replicationTracker = replicationTracker;
328+
this.replicationTracker = new ReplicationTracker(
329+
shardId,
330+
aId,
331+
indexSettings,
332+
primaryTerm,
333+
UNASSIGNED_SEQ_NO,
334+
globalCheckpointListeners::globalCheckpointUpdated,
335+
threadPool::absoluteTimeInMillis,
336+
(retentionLeases, listener) -> retentionLeaseSyncer.sync(shardId, retentionLeases, listener));
340337

341338
// the query cache is a node-level thing, however we want the most popular filters
342339
// to be computed on a per-shard basis
@@ -443,16 +440,17 @@ public void updateShardState(final ShardRouting newRouting,
443440
final ShardRouting currentRouting;
444441
synchronized (mutex) {
445442
currentRouting = this.shardRouting;
443+
assert currentRouting != null;
446444

447445
if (!newRouting.shardId().equals(shardId())) {
448446
throw new IllegalArgumentException("Trying to set a routing entry with shardId " +
449447
newRouting.shardId() + " on a shard with shardId " + shardId());
450448
}
451-
if ((currentRouting == null || newRouting.isSameAllocation(currentRouting)) == false) {
449+
if (newRouting.isSameAllocation(currentRouting) == false) {
452450
throw new IllegalArgumentException("Trying to set a routing entry with a different allocation. Current " +
453451
currentRouting + ", new " + newRouting);
454452
}
455-
if (currentRouting != null && currentRouting.primary() && newRouting.primary() == false) {
453+
if (currentRouting.primary() && newRouting.primary() == false) {
456454
throw new IllegalArgumentException("illegal state: trying to move shard from primary mode to replica mode. Current "
457455
+ currentRouting + ", new " + newRouting);
458456
}
@@ -586,7 +584,7 @@ public void onFailure(Exception e) {
586584
: "a started primary with non-pending operation term must be in primary mode " + this.shardRouting;
587585
shardStateUpdated.countDown();
588586
}
589-
if (currentRouting != null && currentRouting.active() == false && newRouting.active()) {
587+
if (currentRouting.active() == false && newRouting.active()) {
590588
indexEventListener.afterIndexShardStarted(this);
591589
}
592590
if (newRouting.equals(currentRouting) == false) {
@@ -631,8 +629,7 @@ public IndexShardState markAsRecovering(String reason, RecoveryState recoverySta
631629
public void relocated(final String targetAllocationId, final Consumer<ReplicationTracker.PrimaryContext> consumer)
632630
throws IllegalIndexShardStateException, IllegalStateException, InterruptedException {
633631
assert shardRouting.primary() : "only primaries can be marked as relocated: " + shardRouting;
634-
final Releasable forceRefreshes = refreshListeners.forceRefreshes();
635-
try {
632+
try (Releasable forceRefreshes = refreshListeners.forceRefreshes()) {
636633
indexShardOperationPermits.blockOperations(30, TimeUnit.MINUTES, () -> {
637634
forceRefreshes.close();
638635
// no shard operation permits are being held here, move state from started to relocated
@@ -665,8 +662,6 @@ public void relocated(final String targetAllocationId, final Consumer<Replicatio
665662
// Fail primary relocation source and target shards.
666663
failShard("timed out waiting for relocation hand-off to complete", null);
667664
throw new IndexShardClosedException(shardId(), "timed out waiting for relocation hand-off to complete");
668-
} finally {
669-
forceRefreshes.close();
670665
}
671666
}
672667

@@ -745,7 +740,7 @@ private Engine.IndexResult applyIndexOperation(Engine engine, long seqNo, long o
745740
sourceWithResolvedType = new SourceToParse(sourceToParse.index(), resolvedType, sourceToParse.id(),
746741
sourceToParse.source(), sourceToParse.getXContentType(), sourceToParse.routing());
747742
}
748-
operation = prepareIndex(docMapper(resolvedType), indexSettings.getIndexVersionCreated(), sourceWithResolvedType,
743+
operation = prepareIndex(docMapper(resolvedType), sourceWithResolvedType,
749744
seqNo, opPrimaryTerm, version, versionType, origin, autoGeneratedTimeStamp, isRetry, ifSeqNo, ifPrimaryTerm);
750745
Mapping update = operation.parsedDoc().dynamicMappingsUpdate();
751746
if (update != null) {
@@ -763,7 +758,7 @@ private Engine.IndexResult applyIndexOperation(Engine engine, long seqNo, long o
763758
return index(engine, operation);
764759
}
765760

766-
public static Engine.Index prepareIndex(DocumentMapperForType docMapper, Version indexCreatedVersion, SourceToParse source, long seqNo,
761+
public static Engine.Index prepareIndex(DocumentMapperForType docMapper, SourceToParse source, long seqNo,
767762
long primaryTerm, long version, VersionType versionType, Engine.Operation.Origin origin,
768763
long autoGeneratedIdTimestamp, boolean isRetry,
769764
long ifSeqNo, long ifPrimaryTerm) {
@@ -1529,7 +1524,7 @@ private void innerOpenEngineAndTranslog() throws IOException {
15291524
// time elapses after the engine is created above (pulling the config settings) until we set the engine reference, during
15301525
// which settings changes could possibly have happened, so here we forcefully push any config changes to the new engine.
15311526
onSettingsChanged();
1532-
assertSequenceNumbersInCommit();
1527+
assert assertSequenceNumbersInCommit();
15331528
assert recoveryState.getStage() == RecoveryState.Stage.TRANSLOG : "TRANSLOG stage expected but was: " + recoveryState.getStage();
15341529
}
15351530

@@ -1546,7 +1541,7 @@ private boolean assertSequenceNumbersInCommit() throws IOException {
15461541
return true;
15471542
}
15481543

1549-
protected void onNewEngine(Engine newEngine) {
1544+
private void onNewEngine(Engine newEngine) {
15501545
refreshListeners.setCurrentRefreshLocationSupplier(newEngine::getTranslogLastWriteLocation);
15511546
}
15521547

@@ -1858,10 +1853,6 @@ public List<Segment> segments(boolean verbose) {
18581853
return getEngine().segments(verbose);
18591854
}
18601855

1861-
public void flushAndCloseEngine() throws IOException {
1862-
getEngine().flushAndClose();
1863-
}
1864-
18651856
public String getHistoryUUID() {
18661857
return getEngine().getHistoryUUID();
18671858
}
@@ -2876,7 +2867,7 @@ protected void write(List<Tuple<Translog.Location, Consumer<Exception>>> candida
28762867
}
28772868
}
28782869
};
2879-
};
2870+
}
28802871

28812872
/**
28822873
* Syncs the given location with the underlying storage unless already synced. This method might return immediately without
@@ -2988,7 +2979,7 @@ private RefreshListeners buildRefreshListeners() {
29882979
return new RefreshListeners(
29892980
indexSettings::getMaxRefreshListeners,
29902981
() -> refresh("too_many_listeners"),
2991-
threadPool.executor(ThreadPool.Names.LISTENER)::execute,
2982+
threadPool.executor(ThreadPool.Names.LISTENER),
29922983
logger, threadPool.getThreadContext(),
29932984
externalRefreshMetric);
29942985
}

server/src/main/java/org/elasticsearch/index/shard/IndexShardClosedException.java

Lines changed: 0 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -28,10 +28,6 @@ public IndexShardClosedException(ShardId shardId) {
2828
super(shardId, IndexShardState.CLOSED, "Closed");
2929
}
3030

31-
public IndexShardClosedException(ShardId shardId, Throwable t) {
32-
super(shardId, IndexShardState.CLOSED, "Closed", t);
33-
}
34-
3531
public IndexShardClosedException(ShardId shardId, String message) {
3632
super(shardId, IndexShardState.CLOSED, message);
3733
}

server/src/main/java/org/elasticsearch/index/shard/InternalIndexingStats.java

Lines changed: 0 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -176,10 +176,5 @@ IndexingStats.Stats stats(boolean isThrottled, long currentThrottleMillis) {
176176
deleteMetric.count(), TimeUnit.NANOSECONDS.toMillis(deleteMetric.sum()), deleteCurrent.count(),
177177
noopUpdates.count(), isThrottled, TimeUnit.MILLISECONDS.toMillis(currentThrottleMillis));
178178
}
179-
180-
void clear() {
181-
indexMetric.clear();
182-
deleteMetric.clear();
183-
}
184179
}
185180
}

server/src/main/java/org/elasticsearch/index/shard/PrimaryReplicaSyncer.java

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -172,7 +172,7 @@ public void onFailure(Exception e) {
172172
}
173173
};
174174
try {
175-
new SnapshotSender(logger, syncAction, resyncTask, shardId, primaryAllocationId, primaryTerm, snapshot, chunkSize.bytesAsInt(),
175+
new SnapshotSender(syncAction, resyncTask, shardId, primaryAllocationId, primaryTerm, snapshot, chunkSize.bytesAsInt(),
176176
startingSeqNo, maxSeqNo, maxSeenAutoIdTimestamp, wrappedListener).run();
177177
} catch (Exception e) {
178178
wrappedListener.onFailure(e);
@@ -200,12 +200,12 @@ static class SnapshotSender extends AbstractRunnable implements ActionListener<R
200200
private final AtomicBoolean firstMessage = new AtomicBoolean(true);
201201
private final AtomicInteger totalSentOps = new AtomicInteger();
202202
private final AtomicInteger totalSkippedOps = new AtomicInteger();
203-
private AtomicBoolean closed = new AtomicBoolean();
203+
private final AtomicBoolean closed = new AtomicBoolean();
204204

205-
SnapshotSender(Logger logger, SyncAction syncAction, ResyncTask task, ShardId shardId, String primaryAllocationId, long primaryTerm,
205+
SnapshotSender(SyncAction syncAction, ResyncTask task, ShardId shardId, String primaryAllocationId, long primaryTerm,
206206
Translog.Snapshot snapshot, int chunkSizeInBytes, long startingSeqNo, long maxSeqNo,
207207
long maxSeenAutoIdTimestamp, ActionListener<Void> listener) {
208-
this.logger = logger;
208+
this.logger = PrimaryReplicaSyncer.logger;
209209
this.syncAction = syncAction;
210210
this.task = task;
211211
this.shardId = shardId;
@@ -232,7 +232,7 @@ public void onFailure(Exception e) {
232232
}
233233
}
234234

235-
private static Translog.Operation[] EMPTY_ARRAY = new Translog.Operation[0];
235+
private static final Translog.Operation[] EMPTY_ARRAY = new Translog.Operation[0];
236236

237237
@Override
238238
protected void doRun() throws Exception {

server/src/main/java/org/elasticsearch/index/shard/RemoveCorruptedLuceneSegmentsAction.java

Lines changed: 1 addition & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -33,8 +33,7 @@
3333
*/
3434
public class RemoveCorruptedLuceneSegmentsAction {
3535

36-
public Tuple<RemoveCorruptedShardDataCommand.CleanStatus, String> getCleanStatus(ShardPath shardPath,
37-
Directory indexDirectory,
36+
public Tuple<RemoveCorruptedShardDataCommand.CleanStatus, String> getCleanStatus(Directory indexDirectory,
3837
Lock writeLock,
3938
PrintStream printStream,
4039
boolean verbose) throws IOException {
@@ -62,7 +61,6 @@ public Tuple<RemoveCorruptedShardDataCommand.CleanStatus, String> getCleanStatus
6261
}
6362

6463
public void execute(Terminal terminal,
65-
ShardPath shardPath,
6664
Directory indexDirectory,
6765
Lock writeLock,
6866
PrintStream printStream,

server/src/main/java/org/elasticsearch/index/shard/RemoveCorruptedShardDataCommand.java

Lines changed: 4 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -289,7 +289,7 @@ public void write(int b) {
289289
terminal.println("Opening Lucene index at " + indexPath);
290290
terminal.println("");
291291
try {
292-
indexCleanStatus = removeCorruptedLuceneSegmentsAction.getCleanStatus(shardPath, indexDir,
292+
indexCleanStatus = removeCorruptedLuceneSegmentsAction.getCleanStatus(indexDir,
293293
writeIndexLock, printStream, verbose);
294294
} catch (Exception e) {
295295
terminal.println(e.getMessage());
@@ -355,7 +355,7 @@ public void write(int b) {
355355
confirm("Continue and remove corrupted data from the shard ?", terminal);
356356

357357
if (indexStatus != CleanStatus.CLEAN) {
358-
removeCorruptedLuceneSegmentsAction.execute(terminal, shardPath, indexDir,
358+
removeCorruptedLuceneSegmentsAction.execute(terminal, indexDir,
359359
writeIndexLock, printStream, verbose);
360360
}
361361

@@ -373,7 +373,7 @@ public void write(int b) {
373373

374374
// newHistoryCommit obtains its own lock
375375
addNewHistoryCommit(indexDir, terminal, translogStatus != CleanStatus.CLEAN);
376-
newAllocationId(environment, shardPath, terminal);
376+
newAllocationId(shardPath, terminal);
377377
if (indexStatus != CleanStatus.CLEAN) {
378378
dropCorruptMarkerFiles(terminal, indexPath, indexDir, indexStatus == CleanStatus.CLEAN_WITH_CORRUPTED_MARKER);
379379
}
@@ -425,7 +425,7 @@ protected void addNewHistoryCommit(Directory indexDirectory, Terminal terminal,
425425
}
426426
}
427427

428-
protected void newAllocationId(Environment environment, ShardPath shardPath, Terminal terminal) throws IOException {
428+
private void newAllocationId(ShardPath shardPath, Terminal terminal) throws IOException {
429429
final Path shardStatePath = shardPath.getShardStatePath();
430430
final ShardStateMetaData shardStateMetaData =
431431
ShardStateMetaData.FORMAT.loadLatestState(logger, namedXContentRegistry, shardStatePath);

server/src/main/java/org/elasticsearch/index/shard/ShardPath.java

Lines changed: 3 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -34,6 +34,7 @@
3434
import java.util.Arrays;
3535
import java.util.HashMap;
3636
import java.util.Map;
37+
import java.util.Objects;
3738

3839
public final class ShardPath {
3940
public static final String INDEX_FOLDER_NAME = "index";
@@ -283,10 +284,10 @@ public boolean equals(Object o) {
283284
return false;
284285
}
285286
final ShardPath shardPath = (ShardPath) o;
286-
if (shardId != null ? !shardId.equals(shardPath.shardId) : shardPath.shardId != null) {
287+
if (Objects.equals(shardId, shardPath.shardId) == false) {
287288
return false;
288289
}
289-
if (path != null ? !path.equals(shardPath.path) : shardPath.path != null) {
290+
if (Objects.equals(path, shardPath.path) == false) {
290291
return false;
291292
}
292293

0 commit comments

Comments
 (0)