CavalliumDBEngine/src/main/java/it/cavallium/dbengine/database/disk/LLLocalLuceneIndex.java

528 lines
19 KiB
Java
Raw Normal View History

2020-12-07 22:15:18 +01:00
package it.cavallium.dbengine.database.disk;
2021-09-10 12:13:52 +02:00
import static it.cavallium.dbengine.database.LLUtils.MARKER_LUCENE;
import static it.cavallium.dbengine.lucene.searcher.LLSearchTransformer.NO_TRANSFORMATION;
2021-09-10 12:13:52 +02:00
import io.micrometer.core.instrument.MeterRegistry;
2021-11-08 11:17:52 +01:00
import io.net5.buffer.api.Resource;
2021-09-18 18:34:21 +02:00
import io.net5.buffer.api.Send;
2021-11-18 17:13:53 +01:00
import io.net5.buffer.api.internal.ResourceSupport;
2021-07-01 21:19:52 +02:00
import it.cavallium.dbengine.client.DirectIOOptions;
2021-05-28 16:04:59 +02:00
import it.cavallium.dbengine.client.IndicizerAnalyzers;
import it.cavallium.dbengine.client.IndicizerSimilarities;
2021-07-01 21:19:52 +02:00
import it.cavallium.dbengine.client.LuceneOptions;
import it.cavallium.dbengine.client.NRTCachingOptions;
2021-11-19 19:03:31 +01:00
import it.cavallium.dbengine.client.query.QueryParser;
import it.cavallium.dbengine.client.query.current.data.Query;
2021-03-02 01:53:36 +01:00
import it.cavallium.dbengine.client.query.current.data.QueryParams;
2021-11-07 17:46:40 +01:00
import it.cavallium.dbengine.database.LLIndexRequest;
2021-11-07 18:34:34 +01:00
import it.cavallium.dbengine.database.LLSoftUpdateDocument;
2021-11-07 17:46:40 +01:00
import it.cavallium.dbengine.database.LLUpdateDocument;
import it.cavallium.dbengine.database.LLItem;
import it.cavallium.dbengine.database.LLLuceneIndex;
import it.cavallium.dbengine.database.LLSearchResultShard;
import it.cavallium.dbengine.database.LLSnapshot;
import it.cavallium.dbengine.database.LLTerm;
2021-11-07 17:46:40 +01:00
import it.cavallium.dbengine.database.LLUpdateFields;
import it.cavallium.dbengine.database.LLUtils;
import it.cavallium.dbengine.lucene.LuceneHacks;
2021-07-01 21:19:52 +02:00
import it.cavallium.dbengine.lucene.AlwaysDirectIOFSDirectory;
import it.cavallium.dbengine.lucene.LuceneUtils;
2021-11-19 19:03:31 +01:00
import it.cavallium.dbengine.lucene.collector.Buckets;
import it.cavallium.dbengine.lucene.searcher.AdaptiveLocalSearcher;
2021-11-18 17:13:53 +01:00
import it.cavallium.dbengine.lucene.searcher.BucketParams;
import it.cavallium.dbengine.lucene.searcher.DecimalBucketMultiSearcher;
2021-07-06 01:30:37 +02:00
import it.cavallium.dbengine.lucene.searcher.LocalQueryParams;
import it.cavallium.dbengine.lucene.searcher.LocalSearcher;
2021-09-19 19:59:37 +02:00
import it.cavallium.dbengine.lucene.searcher.LLSearchTransformer;
2021-11-18 17:13:53 +01:00
import it.unimi.dsi.fastutil.doubles.DoubleArrayList;
2020-12-07 22:15:18 +01:00
import java.io.IOException;
import java.nio.file.Path;
2021-11-19 19:03:31 +01:00
import java.util.ArrayList;
2021-11-18 17:13:53 +01:00
import java.util.List;
2020-12-07 22:15:18 +01:00
import java.util.Map;
import java.util.Map.Entry;
2020-12-07 22:15:18 +01:00
import java.util.Set;
import java.util.concurrent.Callable;
import java.util.concurrent.ExecutorService;
import java.util.concurrent.Executors;
2021-09-06 18:52:21 +02:00
import java.util.concurrent.Phaser;
2020-12-07 22:15:18 +01:00
import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicBoolean;
import org.apache.lucene.analysis.miscellaneous.PerFieldAnalyzerWrapper;
2021-06-25 20:06:58 +02:00
import org.apache.lucene.index.ConcurrentMergeScheduler;
2020-12-07 22:15:18 +01:00
import org.apache.lucene.index.IndexWriter;
import org.apache.lucene.index.IndexWriterConfig;
import org.apache.lucene.index.KeepOnlyLastCommitDeletionPolicy;
2021-11-20 01:30:06 +01:00
import org.apache.lucene.index.MergePolicy;
2021-07-06 14:33:47 +02:00
import org.apache.lucene.index.MergeScheduler;
2021-06-25 20:06:58 +02:00
import org.apache.lucene.index.SerialMergeScheduler;
2020-12-07 22:15:18 +01:00
import org.apache.lucene.index.SnapshotDeletionPolicy;
2021-11-20 01:30:06 +01:00
import org.apache.lucene.index.TieredMergePolicy;
2021-05-25 01:12:24 +02:00
import org.apache.lucene.misc.store.DirectIODirectory;
import org.apache.lucene.search.similarities.Similarity;
2021-05-25 01:12:24 +02:00
import org.apache.lucene.store.ByteBuffersDirectory;
2020-12-07 22:15:18 +01:00
import org.apache.lucene.store.Directory;
2020-12-31 20:10:47 +01:00
import org.apache.lucene.store.FSDirectory;
import org.apache.lucene.store.MMapDirectory;
2021-05-25 01:12:24 +02:00
import org.apache.lucene.store.NIOFSDirectory;
import org.apache.lucene.store.NRTCachingDirectory;
import org.apache.lucene.util.Constants;
2021-11-19 19:03:31 +01:00
import org.jetbrains.annotations.NotNull;
2020-12-07 22:15:18 +01:00
import org.jetbrains.annotations.Nullable;
import org.warp.commonutils.functional.IORunnable;
2021-02-25 00:00:16 +01:00
import org.warp.commonutils.log.Logger;
import org.warp.commonutils.log.LoggerFactory;
import org.warp.commonutils.type.ShortNamedThreadFactory;
2020-12-07 22:15:18 +01:00
import reactor.core.publisher.Flux;
import reactor.core.publisher.Mono;
import reactor.core.scheduler.Scheduler;
2020-12-07 22:15:18 +01:00
import reactor.core.scheduler.Schedulers;
import reactor.util.function.Tuple2;
2020-12-07 22:15:18 +01:00
public class LLLocalLuceneIndex implements LLLuceneIndex {
2021-02-25 00:00:16 +01:00
protected static final Logger logger = LoggerFactory.getLogger(LLLocalLuceneIndex.class);
private final LocalSearcher localSearcher;
2021-11-18 17:13:53 +01:00
private final DecimalBucketMultiSearcher decimalBucketMultiSearcher = new DecimalBucketMultiSearcher();
/**
* Global lucene index scheduler.
* There is only a single thread globally to not overwhelm the disk with
2021-02-03 14:37:02 +01:00
* concurrent commits or concurrent refreshes.
*/
2021-09-18 18:34:21 +02:00
private static final Scheduler luceneHeavyTasksScheduler = Schedulers.single(Schedulers.boundedElastic());
private static final ExecutorService SAFE_EXECUTOR = Executors.newCachedThreadPool(new ShortNamedThreadFactory("lucene-index-impl"));
private final MeterRegistry meterRegistry;
2020-12-07 22:15:18 +01:00
private final String luceneIndexName;
private final IndexWriter indexWriter;
2021-09-06 15:06:51 +02:00
private final SnapshotsManager snapshotsManager;
2021-09-18 18:34:21 +02:00
private final IndexSearcherManager searcherManager;
private final PerFieldAnalyzerWrapper luceneAnalyzer;
private final Similarity luceneSimilarity;
2020-12-07 22:15:18 +01:00
private final Directory directory;
private final boolean lowMemory;
2021-09-06 18:52:21 +02:00
private final Phaser activeTasks = new Phaser(1);
private final AtomicBoolean closeRequested = new AtomicBoolean();
2021-11-21 12:31:23 +01:00
public LLLocalLuceneIndex(LLTempLMDBEnv env,
@Nullable Path luceneBasePath,
MeterRegistry meterRegistry,
2020-12-07 22:15:18 +01:00
String name,
2021-05-28 16:04:59 +02:00
IndicizerAnalyzers indicizerAnalyzers,
IndicizerSimilarities indicizerSimilarities,
LuceneOptions luceneOptions,
@Nullable LuceneHacks luceneHacks) throws IOException {
this.meterRegistry = meterRegistry;
2021-07-10 20:52:01 +02:00
Path directoryPath;
if (luceneOptions.inMemory() != (luceneBasePath == null)) {
throw new IllegalArgumentException();
} else if (luceneBasePath != null) {
directoryPath = luceneBasePath.resolve(name + ".lucene.db");
} else {
directoryPath = null;
}
2020-12-07 22:15:18 +01:00
if (name.length() == 0) {
throw new IOException("Empty lucene database name");
}
2021-05-25 01:12:24 +02:00
if (!MMapDirectory.UNMAP_SUPPORTED) {
logger.error("Unmap is unsupported, lucene will run slower: {}", MMapDirectory.UNMAP_NOT_SUPPORTED_REASON);
} else {
logger.debug("Lucene MMap is supported");
}
2021-07-01 21:19:52 +02:00
boolean lowMemory = luceneOptions.lowMemory();
if (luceneOptions.inMemory()) {
2021-05-25 01:12:24 +02:00
this.directory = new ByteBuffersDirectory();
} else {
Directory directory;
{
2021-07-01 21:19:52 +02:00
Directory forcedDirectFsDirectory = null;
if (luceneOptions.directIOOptions().isPresent()) {
DirectIOOptions directIOOptions = luceneOptions.directIOOptions().get();
if (directIOOptions.alwaysForceDirectIO()) {
try {
forcedDirectFsDirectory = new AlwaysDirectIOFSDirectory(directoryPath);
} catch (UnsupportedOperationException ex) {
logger.warn("Failed to open FSDirectory with DIRECT flag", ex);
}
}
}
if (forcedDirectFsDirectory != null) {
directory = forcedDirectFsDirectory;
} else {
FSDirectory fsDirectory;
if (luceneOptions.allowMemoryMapping()) {
fsDirectory = FSDirectory.open(directoryPath);
2021-05-25 01:12:24 +02:00
} else {
2021-07-01 21:19:52 +02:00
fsDirectory = new NIOFSDirectory(directoryPath);
}
if (Constants.LINUX || Constants.MAC_OS_X) {
try {
int mergeBufferSize;
long minBytesDirect;
if (luceneOptions.directIOOptions().isPresent()) {
var directIOOptions = luceneOptions.directIOOptions().get();
mergeBufferSize = directIOOptions.mergeBufferSize();
minBytesDirect = directIOOptions.minBytesDirect();
} else {
mergeBufferSize = DirectIODirectory.DEFAULT_MERGE_BUFFER_SIZE;
minBytesDirect = DirectIODirectory.DEFAULT_MIN_BYTES_DIRECT;
}
directory = new DirectIODirectory(fsDirectory, mergeBufferSize, minBytesDirect);
} catch (UnsupportedOperationException ex) {
logger.warn("Failed to open FSDirectory with DIRECT flag", ex);
directory = fsDirectory;
}
} else {
directory = fsDirectory;
2021-05-25 01:12:24 +02:00
}
}
}
2021-07-01 21:19:52 +02:00
if (luceneOptions.nrtCachingOptions().isPresent()) {
NRTCachingOptions nrtCachingOptions = luceneOptions.nrtCachingOptions().get();
2021-09-18 18:34:21 +02:00
directory = new NRTCachingDirectory(directory, nrtCachingOptions.maxMergeSizeMB(),
nrtCachingOptions.maxCachedMB());
2021-05-25 01:12:24 +02:00
}
this.directory = directory;
}
2021-06-06 02:23:51 +02:00
2020-12-07 22:15:18 +01:00
this.luceneIndexName = name;
2021-09-06 15:06:51 +02:00
var snapshotter = new SnapshotDeletionPolicy(new KeepOnlyLastCommitDeletionPolicy());
2020-12-07 22:15:18 +01:00
this.lowMemory = lowMemory;
this.luceneAnalyzer = LuceneUtils.toPerFieldAnalyzerWrapper(indicizerAnalyzers);
this.luceneSimilarity = LuceneUtils.toPerFieldSimilarityWrapper(indicizerSimilarities);
if (luceneHacks != null && luceneHacks.customLocalSearcher() != null) {
localSearcher = luceneHacks.customLocalSearcher().get();
} else {
2021-11-21 12:31:23 +01:00
localSearcher = new AdaptiveLocalSearcher(env);
}
2021-06-06 02:23:51 +02:00
var indexWriterConfig = new IndexWriterConfig(luceneAnalyzer);
2020-12-07 22:15:18 +01:00
indexWriterConfig.setOpenMode(IndexWriterConfig.OpenMode.CREATE_OR_APPEND);
indexWriterConfig.setIndexDeletionPolicy(snapshotter);
indexWriterConfig.setCommitOnClose(true);
int writerSchedulerMaxThreadCount;
2021-07-06 14:33:47 +02:00
MergeScheduler mergeScheduler;
2020-12-07 22:15:18 +01:00
if (lowMemory) {
2021-07-06 14:33:47 +02:00
mergeScheduler = new SerialMergeScheduler();
writerSchedulerMaxThreadCount = 1;
2020-12-07 22:15:18 +01:00
} else {
2021-07-06 14:33:47 +02:00
var concurrentMergeScheduler = new ConcurrentMergeScheduler();
2021-07-23 15:20:33 +02:00
concurrentMergeScheduler.setDefaultMaxMergesAndThreads(false);
if (luceneOptions.inMemory()) {
concurrentMergeScheduler.disableAutoIOThrottle();
} else {
concurrentMergeScheduler.enableAutoIOThrottle();
}
writerSchedulerMaxThreadCount = concurrentMergeScheduler.getMaxThreadCount();
2021-07-06 14:33:47 +02:00
mergeScheduler = concurrentMergeScheduler;
2020-12-07 22:15:18 +01:00
}
logger.trace("WriterSchedulerMaxThreadCount: {}", writerSchedulerMaxThreadCount);
2021-07-06 14:33:47 +02:00
indexWriterConfig.setMergeScheduler(mergeScheduler);
if (luceneOptions.indexWriterBufferSize() == -1) {
//todo: allow to configure maxbuffereddocs fallback
indexWriterConfig.setMaxBufferedDocs(1000);
// disable ram buffer size after enabling maxBufferedDocs
indexWriterConfig.setRAMBufferSizeMB(-1);
} else {
indexWriterConfig.setRAMBufferSizeMB(luceneOptions.indexWriterBufferSize() / 1024D / 1024D);
}
2021-07-06 14:33:47 +02:00
indexWriterConfig.setReaderPooling(false);
indexWriterConfig.setSimilarity(getLuceneSimilarity());
2020-12-07 22:15:18 +01:00
this.indexWriter = new IndexWriter(directory, indexWriterConfig);
2021-09-06 18:52:21 +02:00
this.snapshotsManager = new SnapshotsManager(indexWriter, snapshotter);
2021-09-06 18:24:36 +02:00
this.searcherManager = new CachedIndexSearcherManager(indexWriter,
2021-09-06 15:06:51 +02:00
snapshotsManager,
getLuceneSimilarity(),
2021-07-06 14:33:47 +02:00
luceneOptions.applyAllDeletes(),
luceneOptions.writeAllDeletes(),
2021-09-06 15:06:51 +02:00
luceneOptions.queryRefreshDebounceTime()
2021-07-06 14:33:47 +02:00
);
// Start scheduled tasks
2021-09-06 18:52:21 +02:00
var commitMillis = luceneOptions.commitDebounceTime().toMillis();
luceneHeavyTasksScheduler.schedulePeriodically(this::scheduledCommit, commitMillis, commitMillis,
TimeUnit.MILLISECONDS);
}
private Similarity getLuceneSimilarity() {
return luceneSimilarity;
}
2020-12-07 22:15:18 +01:00
@Override
public String getLuceneIndexName() {
return luceneIndexName;
}
@Override
public Mono<LLSnapshot> takeSnapshot() {
return snapshotsManager.takeSnapshot().subscribeOn(luceneHeavyTasksScheduler).transform(this::ensureOpen);
}
private <V> Mono<V> ensureOpen(Mono<V> mono) {
return Mono.<Void>fromCallable(() -> {
if (closeRequested.get()) {
throw new IllegalStateException("Lucene index is closed");
} else {
return null;
}
}).then(mono).doFirst(activeTasks::register).doFinally(s -> activeTasks.arriveAndDeregister());
}
private <V> Mono<V> runSafe(Callable<V> callable) {
return Mono.<V>create(sink -> {
var future = SAFE_EXECUTOR.submit(() -> {
try {
var result = callable.call();
if (result != null) {
sink.success(result);
} else {
sink.success();
}
} catch (Throwable e) {
sink.error(e);
}
});
sink.onDispose(() -> future.cancel(false));
});
}
private <V> Mono<V> runSafe(IORunnable runnable) {
return Mono.create(sink -> {
var future = SAFE_EXECUTOR.submit(() -> {
try {
runnable.run();
sink.success();
} catch (Throwable e) {
sink.error(e);
}
});
sink.onDispose(() -> future.cancel(false));
});
2020-12-07 22:15:18 +01:00
}
@Override
public Mono<Void> releaseSnapshot(LLSnapshot snapshot) {
2021-09-06 15:06:51 +02:00
return snapshotsManager.releaseSnapshot(snapshot);
2020-12-07 22:15:18 +01:00
}
@Override
2021-11-07 17:46:40 +01:00
public Mono<Void> addDocument(LLTerm key, LLUpdateDocument doc) {
return this.<Void>runSafe(() -> indexWriter.addDocument(LLUtils.toDocument(doc))).transform(this::ensureOpen);
2020-12-07 22:15:18 +01:00
}
@Override
2021-11-07 17:46:40 +01:00
public Mono<Void> addDocuments(Flux<Entry<LLTerm, LLUpdateDocument>> documents) {
return documents
2021-05-28 16:04:59 +02:00
.collectList()
.flatMap(documentsList -> this.<Void>runSafe(() -> indexWriter.addDocuments(LLUtils
.toDocumentsFromEntries(documentsList))))
.transform(this::ensureOpen);
2020-12-07 22:15:18 +01:00
}
2020-12-07 22:15:18 +01:00
@Override
public Mono<Void> deleteDocument(LLTerm id) {
return this.<Void>runSafe(() -> indexWriter.deleteDocuments(LLUtils.toTerm(id))).transform(this::ensureOpen);
2020-12-07 22:15:18 +01:00
}
@Override
2021-11-07 17:46:40 +01:00
public Mono<Void> update(LLTerm id, LLIndexRequest request) {
return this
2021-11-07 17:46:40 +01:00
.<Void>runSafe(() -> {
if (request instanceof LLUpdateDocument updateDocument) {
indexWriter.updateDocument(LLUtils.toTerm(id), LLUtils.toDocument(updateDocument));
2021-11-07 18:34:34 +01:00
} else if (request instanceof LLSoftUpdateDocument softUpdateDocument) {
indexWriter.softUpdateDocument(LLUtils.toTerm(id),
LLUtils.toDocument(softUpdateDocument.items()),
LLUtils.toFields(softUpdateDocument.softDeleteItems())
);
2021-11-07 17:46:40 +01:00
} else if (request instanceof LLUpdateFields updateFields) {
indexWriter.updateDocValues(LLUtils.toTerm(id), LLUtils.toFields(updateFields.items()));
} else {
throw new UnsupportedOperationException("Unexpected request type: " + request);
}
})
.transform(this::ensureOpen);
2020-12-07 22:15:18 +01:00
}
@Override
2021-11-07 17:46:40 +01:00
public Mono<Void> updateDocuments(Mono<Map<LLTerm, LLUpdateDocument>> documents) {
return documents.flatMap(this::updateDocuments).then();
}
2021-11-07 17:46:40 +01:00
private Mono<Void> updateDocuments(Map<LLTerm, LLUpdateDocument> documentsMap) {
return this.<Void>runSafe(() -> {
2021-11-07 17:46:40 +01:00
for (Entry<LLTerm, LLUpdateDocument> entry : documentsMap.entrySet()) {
LLTerm key = entry.getKey();
2021-11-07 17:46:40 +01:00
LLUpdateDocument value = entry.getValue();
indexWriter.updateDocument(LLUtils.toTerm(key), LLUtils.toDocument(value));
}
}).transform(this::ensureOpen);
2020-12-07 22:15:18 +01:00
}
@Override
public Mono<Void> deleteAll() {
return this.<Void>runSafe(() -> {
indexWriter.deleteAll();
indexWriter.forceMergeDeletes(true);
indexWriter.commit();
}).subscribeOn(luceneHeavyTasksScheduler).transform(this::ensureOpen);
2020-12-07 22:15:18 +01:00
}
@Override
2021-11-08 11:17:52 +01:00
public Mono<LLSearchResultShard> moreLikeThis(@Nullable LLSnapshot snapshot,
2021-03-02 01:53:36 +01:00
QueryParams queryParams,
String keyFieldName,
Flux<Tuple2<String, Set<String>>> mltDocumentFieldsFlux) {
2021-11-16 23:19:23 +01:00
LocalQueryParams localQueryParams = LuceneUtils.toLocalQueryParams(queryParams, luceneAnalyzer);
var searcher = this.searcherManager.retrieveSearcher(snapshot);
var transformer = new MoreLikeThisTransformer(mltDocumentFieldsFlux);
2021-07-06 01:30:37 +02:00
2021-11-08 11:17:52 +01:00
return localSearcher
.collect(searcher, localQueryParams, keyFieldName, transformer)
.map(result -> new LLSearchResultShard(result.results(), result.totalHitsCount(), result::close))
.doOnDiscard(Send.class, Send::close)
.doOnDiscard(Resource.class, Resource::close);
2021-07-06 01:30:37 +02:00
}
2020-12-07 22:15:18 +01:00
@Override
2021-11-08 11:17:52 +01:00
public Mono<LLSearchResultShard> search(@Nullable LLSnapshot snapshot, QueryParams queryParams,
2021-09-18 18:34:21 +02:00
String keyFieldName) {
2021-11-16 23:19:23 +01:00
LocalQueryParams localQueryParams = LuceneUtils.toLocalQueryParams(queryParams, luceneAnalyzer);
var searcher = searcherManager.retrieveSearcher(snapshot);
2021-11-08 11:17:52 +01:00
return localSearcher
.collect(searcher, localQueryParams, keyFieldName, NO_TRANSFORMATION)
.map(result -> new LLSearchResultShard(result.results(), result.totalHitsCount(), result::close))
2021-11-08 12:06:32 +01:00
.doOnDiscard(Send.class, Send::close)
.doOnDiscard(Resource.class, Resource::close);
2021-11-18 17:13:53 +01:00
}
@Override
2021-11-19 19:03:31 +01:00
public Mono<Buckets> computeBuckets(@Nullable LLSnapshot snapshot,
@NotNull List<Query> queries,
@Nullable Query normalizationQuery,
2021-11-18 17:13:53 +01:00
BucketParams bucketParams) {
2021-11-19 19:03:31 +01:00
List<org.apache.lucene.search.Query> localQueries = new ArrayList<>(queries.size());
for (Query query : queries) {
localQueries.add(QueryParser.toQuery(query, luceneAnalyzer));
}
var localNormalizationQuery = QueryParser.toQuery(normalizationQuery, luceneAnalyzer);
2021-11-18 17:13:53 +01:00
var searchers = searcherManager
.retrieveSearcher(snapshot)
.map(indexSearcher -> LLIndexSearchers.unsharded(indexSearcher).send());
return decimalBucketMultiSearcher
2021-11-19 19:03:31 +01:00
.collectMulti(searchers, bucketParams, localQueries, localNormalizationQuery)
2021-11-18 17:13:53 +01:00
.doOnDiscard(Send.class, Send::close)
.doOnDiscard(Resource.class, Resource::close);
}
public Mono<Send<LLIndexSearcher>> retrieveSearcher(@Nullable LLSnapshot snapshot) {
2021-09-18 18:34:21 +02:00
return searcherManager
.retrieveSearcher(snapshot)
2021-11-08 12:06:32 +01:00
.doOnDiscard(Send.class, Send::close)
.doOnDiscard(Resource.class, Resource::close);
2020-12-07 22:15:18 +01:00
}
@Override
public Mono<Void> close() {
return Mono
.<Void>fromCallable(() -> {
2021-09-07 02:36:11 +02:00
logger.info("Waiting IndexWriter tasks...");
2021-09-06 18:52:21 +02:00
activeTasks.arriveAndAwaitAdvance();
2021-09-07 02:36:11 +02:00
logger.info("IndexWriter tasks ended");
2021-09-06 15:06:51 +02:00
return null;
})
.subscribeOn(luceneHeavyTasksScheduler)
.then(searcherManager.close())
.then(Mono.<Void>fromCallable(() -> {
2021-09-07 02:36:11 +02:00
logger.info("Closing IndexWriter...");
2021-02-03 14:37:02 +01:00
//noinspection BlockingMethodInNonBlockingContext
indexWriter.close();
2021-02-03 14:37:02 +01:00
//noinspection BlockingMethodInNonBlockingContext
directory.close();
2021-09-07 02:36:11 +02:00
logger.info("IndexWriter closed");
return null;
}).subscribeOn(luceneHeavyTasksScheduler))
// Avoid closing multiple times
.transformDeferred(mono -> {
if (this.closeRequested.compareAndSet(false, true)) {
logger.trace("Set closeRequested to true. Further update/write calls will result in an error");
return mono;
} else {
logger.debug("Tried to close more than once");
return Mono.empty();
}
});
2020-12-07 22:15:18 +01:00
}
2021-02-03 13:48:30 +01:00
@Override
public Mono<Void> flush() {
return Mono
.<Void>fromCallable(() -> {
if (activeTasks.isTerminated()) return null;
//noinspection BlockingMethodInNonBlockingContext
2021-11-20 01:30:06 +01:00
indexWriter.flush();
2021-02-03 13:48:30 +01:00
return null;
})
.subscribeOn(luceneHeavyTasksScheduler)
.transform(this::ensureOpen);
2021-02-03 13:48:30 +01:00
}
@Override
2021-07-18 19:37:24 +02:00
public Mono<Void> refresh(boolean force) {
2021-02-03 13:48:30 +01:00
return Mono
.<Void>fromCallable(() -> {
2021-09-06 18:52:21 +02:00
activeTasks.register();
2021-02-03 13:48:30 +01:00
try {
2021-09-06 18:52:21 +02:00
if (activeTasks.isTerminated()) return null;
2021-07-18 19:37:24 +02:00
if (force) {
//noinspection BlockingMethodInNonBlockingContext
searcherManager.maybeRefreshBlocking();
} else {
//noinspection BlockingMethodInNonBlockingContext
searcherManager.maybeRefresh();
}
2021-02-03 13:48:30 +01:00
} finally {
2021-09-06 18:52:21 +02:00
activeTasks.arriveAndDeregister();
2021-02-03 13:48:30 +01:00
}
return null;
})
2021-03-03 10:57:45 +01:00
.subscribeOn(luceneHeavyTasksScheduler);
2021-02-03 13:48:30 +01:00
}
2020-12-07 22:15:18 +01:00
private void scheduledCommit() {
try {
2021-11-20 01:30:06 +01:00
indexWriter.commit();
2020-12-07 22:15:18 +01:00
} catch (IOException ex) {
2021-09-10 12:13:52 +02:00
logger.error(MARKER_LUCENE, "Failed to execute a scheduled commit", ex);
2020-12-07 22:15:18 +01:00
}
}
@Override
public boolean isLowMemoryMode() {
return lowMemory;
}
private class MoreLikeThisTransformer implements LLSearchTransformer {
private final Flux<Tuple2<String, Set<String>>> mltDocumentFieldsFlux;
public MoreLikeThisTransformer(Flux<Tuple2<String, Set<String>>> mltDocumentFieldsFlux) {
this.mltDocumentFieldsFlux = mltDocumentFieldsFlux;
}
@Override
public Mono<LocalQueryParams> transform(Mono<TransformerInput> inputMono) {
return inputMono.flatMap(input -> LuceneUtils.getMoreLikeThisQuery(input.indexSearchers(), input.queryParams(),
luceneAnalyzer, luceneSimilarity, mltDocumentFieldsFlux));
}
}
2020-12-07 22:15:18 +01:00
}