Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
Show all changes
34 commits
Select commit Hold shift + click to select a range
8bf64ae
Add file extension metadata to CacheMissCounter
tteofili Sep 9, 2025
a7a21be
record file extensions
tteofili Sep 9, 2025
7eb609f
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 9, 2025
60866ef
[CI] Auto commit changes from spotless
Sep 9, 2025
727c25c
better account for compound / stateless files
tteofili Sep 10, 2025
46c77e3
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 10, 2025
3a8c423
edge case handling
tteofili Sep 10, 2025
764d63c
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 11, 2025
ce78e19
get rid of resource string parsing for now
tteofili Sep 11, 2025
af291ae
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 11, 2025
9116e7e
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 12, 2025
f6951c3
test
tteofili Sep 12, 2025
de40258
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 12, 2025
dbc0f01
test
tteofili Sep 12, 2025
697927c
forbidden apis fixes
tteofili Sep 12, 2025
bd9e5f4
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 12, 2025
fa05d4e
add todos
tteofili Sep 12, 2025
b26a110
Update docs/changelog/134374.yaml
tteofili Sep 12, 2025
ebf6aad
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 12, 2025
4946785
Merge branch 'cachemiss_metric_filext' of github.com:tteofili/elastic…
tteofili Sep 12, 2025
aa84ca3
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 15, 2025
ac9e6ca
just catch Exception
tteofili Sep 15, 2025
201f0dd
avoid asserting code block
tteofili Sep 15, 2025
7978595
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 15, 2025
91461d3
minor fix
tteofili Sep 15, 2025
e987861
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 15, 2025
9ae04c3
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 16, 2025
c651dc5
simplification, more tests
tteofili Sep 16, 2025
be8efcd
spotless
tteofili Sep 16, 2025
6823674
minor test fix
tteofili Sep 16, 2025
0683721
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 16, 2025
3710a77
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 16, 2025
b628553
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 17, 2025
71d1eab
Merge branch 'main' of github.com:elastic/elasticsearch into cachemis…
tteofili Sep 17, 2025
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
5 changes: 5 additions & 0 deletions docs/changelog/134374.yaml
Original file line number Diff line number Diff line change
@@ -0,0 +1,5 @@
pr: 134374
summary: Add file extension metadata to cache miss counter from `SharedBlobCacheService`
area: Search
type: enhancement
issues: []
Original file line number Diff line number Diff line change
Expand Up @@ -9,10 +9,13 @@

package org.elasticsearch.index.store;

import org.apache.lucene.index.IndexFileNames;
import org.elasticsearch.core.Assertions;
import org.elasticsearch.core.SuppressForbidden;
import org.elasticsearch.test.ESTestCase;

import java.util.Locale;

import static org.hamcrest.Matchers.containsString;

public class LuceneFilesExtensionsTests extends ESTestCase {
Expand All @@ -21,6 +24,7 @@ public void testUnknownFileExtension() {
if (Assertions.ENABLED) {
AssertionError e = expectThrows(AssertionError.class, () -> LuceneFilesExtensions.fromExtension("abc"));
assertThat(e.getMessage(), containsString("unknown Lucene file extension [abc]"));
assertFalse(LuceneFilesExtensions.isLuceneExtension("abc"));

setEsAllowUnknownLuceneFileExtensions("true");
try {
Expand All @@ -41,4 +45,19 @@ public void setEsAllowUnknownLuceneFileExtensions(final String value) {
System.setProperty("es.allow_unknown_lucene_file_extensions", value);
}
}

public void testIsLuceneExtension() {
assertFalse(LuceneFilesExtensions.isLuceneExtension(null));
assertFalse(LuceneFilesExtensions.isLuceneExtension("bcde"));
String randomStringWithLuceneExtension = randomAlphanumericOfLength(10)
+ "."
+ LuceneFilesExtensions.values()[randomInt(LuceneFilesExtensions.values().length) - 1].getExtension();
String extension = IndexFileNames.getExtension(randomStringWithLuceneExtension);
assertTrue(extension + " should be considered a Lucene extension", LuceneFilesExtensions.isLuceneExtension(extension));
String upperCaseExtension = extension.toUpperCase(Locale.ROOT);
assertFalse(
upperCaseExtension + " (uppercase) should not be considered a Lucene extension",
LuceneFilesExtensions.isLuceneExtension(upperCaseExtension)
);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -165,6 +165,11 @@ public static LuceneFilesExtensions fromExtension(String ext) {
return null;
}

@Nullable
public static boolean isLuceneExtension(String ext) {
return extensions.containsKey(ext);
Comment on lines +168 to +170
Copy link
Contributor

Choose a reason for hiding this comment

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

nit: this is not nullable

Copy link
Contributor Author

Choose a reason for hiding this comment

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

if I'm not mistaken, IndexFileNames#getExtension might return null in case the fileName doesn't contain a ., that's why I had put the @Nullable annotation.

Copy link
Contributor

@andreidan andreidan Sep 17, 2025

Choose a reason for hiding this comment

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

Oh yes. I see what you mean.

We should @Nullable the ext method parameter ++

Being on the method it indicates the method can return null.

}

@Nullable
public static LuceneFilesExtensions fromFile(String fileName) {
return fromExtension(IndexFileNames.getExtension(fileName));
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -9,6 +9,7 @@

import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
import org.apache.lucene.index.IndexFileNames;
import org.apache.lucene.store.AlreadyClosedException;
import org.elasticsearch.action.ActionListener;
import org.elasticsearch.action.ActionRunnable;
Expand Down Expand Up @@ -37,6 +38,7 @@
import org.elasticsearch.core.TimeValue;
import org.elasticsearch.env.Environment;
import org.elasticsearch.env.NodeEnvironment;
import org.elasticsearch.index.store.LuceneFilesExtensions;
import org.elasticsearch.monitor.fs.FsProbe;
import org.elasticsearch.node.NodeRoleSettings;
import org.elasticsearch.threadpool.ThreadPool;
Expand Down Expand Up @@ -67,6 +69,9 @@
import java.util.function.Predicate;
import java.util.stream.Collectors;

import static org.elasticsearch.blobcache.BlobCacheMetrics.LUCENE_FILE_EXTENSION_ATTRIBUTE_KEY;
import static org.elasticsearch.blobcache.BlobCacheMetrics.NON_LUCENE_EXTENSION_TO_RECORD;

/**
* A caching layer on a local node to minimize network roundtrips to the remote blob store.
*/
Expand Down Expand Up @@ -1254,7 +1259,8 @@ public int populateAndRead(
final ByteRange rangeToWrite,
final ByteRange rangeToRead,
final RangeAvailableHandler reader,
final RangeMissingHandler writer
final RangeMissingHandler writer,
String resourceDescription
) throws Exception {
// some cache files can grow after being created, so rangeToWrite can be larger than the initial {@code length}
assert rangeToWrite.start() >= 0 : rangeToWrite;
Expand All @@ -1273,6 +1279,7 @@ public void fillCacheRange(
IntConsumer progressUpdater,
ActionListener<Void> completionListener
) throws IOException {
String blobFileExtension = getFileExtension(resourceDescription);
writer.fillCacheRange(
channel,
channelPos,
Expand All @@ -1283,7 +1290,8 @@ public void fillCacheRange(
completionListener.map(unused -> {
var elapsedTime = TimeUnit.NANOSECONDS.toMillis(relativeTimeInNanosSupplier.getAsLong() - startTime);
blobCacheMetrics.getCacheMissLoadTimes().record(elapsedTime);
blobCacheMetrics.getCacheMissCounter().increment();
blobCacheMetrics.getCacheMissCounter()
.incrementBy(1L, Map.of(LUCENE_FILE_EXTENSION_ATTRIBUTE_KEY, blobFileExtension));
return null;
})
);
Expand Down Expand Up @@ -2075,4 +2083,17 @@ public void close() {
}
}
}

private static String getFileExtension(String resourceDescription) {
// TODO: consider introspecting resourceDescription for compound files
if (resourceDescription.endsWith(LuceneFilesExtensions.CFS.getExtension())) {
return LuceneFilesExtensions.CFS.getExtension();
}
String extension = IndexFileNames.getExtension(resourceDescription);
if (LuceneFilesExtensions.isLuceneExtension(extension)) {
return extension;
} else {
return NON_LUCENE_EXTENSION_TO_RECORD;
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -43,6 +43,9 @@

import java.io.IOException;
import java.io.InputStream;
import java.nio.ByteBuffer;
import java.nio.file.Files;
import java.nio.file.Path;
import java.util.Collection;
import java.util.HashMap;
import java.util.HashSet;
Expand All @@ -51,6 +54,8 @@
import java.util.Set;
import java.util.concurrent.BrokenBarrierException;
import java.util.concurrent.CyclicBarrier;
import java.util.concurrent.ExecutorService;
import java.util.concurrent.Executors;
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicBoolean;
import java.util.concurrent.atomic.AtomicInteger;
Expand Down Expand Up @@ -171,6 +176,78 @@ public void testBasicEviction() throws IOException {
}
}

public void testCacheMissOnPopulate() throws Exception {
Settings settings = Settings.builder()
.put(NODE_NAME_SETTING.getKey(), "node")
.put(SharedBlobCacheService.SHARED_CACHE_SIZE_SETTING.getKey(), ByteSizeValue.ofBytes(size(50)).getStringRep())
.put(SharedBlobCacheService.SHARED_CACHE_REGION_SIZE_SETTING.getKey(), ByteSizeValue.ofBytes(size(10)).getStringRep())
.put("path.home", createTempDir())
.build();
final DeterministicTaskQueue taskQueue = new DeterministicTaskQueue();
RecordingMeterRegistry recordingMeterRegistry = new RecordingMeterRegistry();
BlobCacheMetrics metrics = new BlobCacheMetrics(recordingMeterRegistry);
ExecutorService ioExecutor = Executors.newCachedThreadPool();
try (
NodeEnvironment environment = new NodeEnvironment(settings, TestEnvironment.newEnvironment(settings));
var cacheService = new SharedBlobCacheService<>(environment, settings, taskQueue.getThreadPool(), ioExecutor, metrics)
) {
ByteRange rangeRead = ByteRange.of(0L, 1L);
ByteRange rangeWrite = ByteRange.of(0L, 1L);
Path tempFile = createTempFile("test", "other");
String resourceDescription = tempFile.toAbsolutePath().toString();
final var cacheKey = generateCacheKey();
SharedBlobCacheService<Object>.CacheFile cacheFile = cacheService.getCacheFile(cacheKey, 1L);

ByteBuffer writeBuffer = ByteBuffer.allocate(1);

final int bytesRead = cacheFile.populateAndRead(
rangeRead,
rangeWrite,
(channel, pos, relativePos, len) -> len,
(channel, channelPos, streamFactory, relativePos, len, progressUpdater, completionListener) -> {
try (var in = Files.newInputStream(tempFile)) {
SharedBytes.copyToCacheFileAligned(channel, in, channelPos, progressUpdater, writeBuffer.clear());
}
ActionListener.completeWith(completionListener, () -> null);
},
resourceDescription
);
assertThat(bytesRead, is(1));
List<Measurement> measurements = recordingMeterRegistry.getRecorder()
.getMeasurements(InstrumentType.LONG_COUNTER, "es.blob_cache.miss_that_triggered_read.total");
Measurement first = measurements.getFirst();
assertThat(first.attributes().get("file_extension"), is("other"));
assertThat(first.value(), is(1L));

Path tempFile2 = createTempFile("test", "cfs");
resourceDescription = tempFile2.toAbsolutePath().toString();
cacheFile = cacheService.getCacheFile(generateCacheKey(), 1L);

ByteBuffer writeBuffer2 = ByteBuffer.allocate(1);

final int bytesRead2 = cacheFile.populateAndRead(
rangeRead,
rangeWrite,
(channel, pos, relativePos, len) -> len,
(channel, channelPos, streamFactory, relativePos, len, progressUpdater, completionListener) -> {
try (var in = Files.newInputStream(tempFile2)) {
SharedBytes.copyToCacheFileAligned(channel, in, channelPos, progressUpdater, writeBuffer2.clear());
}
ActionListener.completeWith(completionListener, () -> null);
},
resourceDescription
);
assertThat(bytesRead2, is(1));

measurements = recordingMeterRegistry.getRecorder()
.getMeasurements(InstrumentType.LONG_COUNTER, "es.blob_cache.miss_that_triggered_read.total");
Measurement measurement = measurements.get(1);
assertThat(measurement.attributes().get("file_extension"), is("cfs"));
assertThat(measurement.value(), is(1L));
}
ioExecutor.shutdown();
}

private static boolean tryEvict(SharedBlobCacheService.CacheFileRegion<Object> region1) {
if (randomBoolean()) {
return region1.tryEvict();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -177,7 +177,8 @@ private void readWithoutBlobCacheSlow(ByteBuffer b, long position, int length) t
return null;
}
}
)
),
fileInfo.physicalName()
);
assert bytesRead == length : bytesRead + " vs " + length;
byteBufferReference.finish(bytesRead);
Expand Down