From d5a87afc2ee830a1fff365020bdc88850d9498d4 Mon Sep 17 00:00:00 2001 From: Anshu Agarwal Date: Sat, 28 Jan 2023 18:53:06 +0530 Subject: [PATCH 01/20] Fix changelog entry, move to 2.x section (#6057) Signed-off-by: Anshu Agarwal --- CHANGELOG.md | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index 99e2606ae22a0..c3beb5a2d3f20 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -10,7 +10,6 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Add getter for path field in NestedQueryBuilder ([#4636](https://github.com/opensearch-project/OpenSearch/pull/4636)) - Allow mmap to use new JDK-19 preview APIs in Apache Lucene 9.4+ ([#5151](https://github.com/opensearch-project/OpenSearch/pull/5151)) - Add support for ppc64le architecture ([#5459](https://github.com/opensearch-project/OpenSearch/pull/5459)) -- Add support to disallow search request with preference parameter with strict weighted shard routing([#5874](https://github.com/opensearch-project/OpenSearch/pull/5874)) ### Dependencies - Bumps `log4j-core` from 2.18.0 to 2.19.0 @@ -80,6 +79,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Add update-index-settings allowlist for searchable snapshot ([#5907](https://github.com/opensearch-project/OpenSearch/pull/5907)) - Replace latches with CompletableFutures for extensions ([#5646](https://github.com/opensearch-project/OpenSearch/pull/5646)) - Add GeoTile and GeoHash Grid aggregations on GeoShapes. ([#5589](https://github.com/opensearch-project/OpenSearch/pull/5589)) +- Add support to disallow search request with preference parameter with strict weighted shard routing([#5874](https://github.com/opensearch-project/OpenSearch/pull/5874)) ### Dependencies - Update nebula-publishing-plugin to 19.2.0 ([#5704](https://github.com/opensearch-project/OpenSearch/pull/5704)) From af566e156fefdba192343ba8f7fce84f17d2a07a Mon Sep 17 00:00:00 2001 From: Ashish Date: Sun, 29 Jan 2023 09:57:22 +0530 Subject: [PATCH 02/20] Batch translog sync/upload per x ms for remote-backed indexes (#5854) * Batch translog upload per x ms to allow high index throughput Signed-off-by: Ashish Singh Co-authored-by: Ashwin Pankaj Co-authored-by: Laxman Muttineni --- .../cluster/metadata/IndexMetadata.java | 42 +++ .../common/settings/IndexScopedSettings.java | 3 +- .../opensearch/common/settings/Setting.java | 17 ++ .../util/concurrent/AsyncIOProcessor.java | 42 ++- .../concurrent/BufferedAsyncIOProcessor.java | 93 +++++++ .../org/opensearch/index/IndexSettings.java | 14 + .../opensearch/index/shard/IndexShard.java | 53 +++- .../org/opensearch/threadpool/ThreadPool.java | 3 + .../BufferedAsyncIOProcessorTests.java | 252 ++++++++++++++++++ .../opensearch/index/IndexSettingsTests.java | 43 +++ ...overyWithRemoteTranslogOnPrimaryTests.java | 1 + ...teStorePeerRecoverySourceHandlerTests.java | 1 + .../threadpool/ScalingThreadPoolTests.java | 1 + 13 files changed, 541 insertions(+), 24 deletions(-) create mode 100644 server/src/main/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessor.java create mode 100644 server/src/test/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessorTests.java diff --git a/server/src/main/java/org/opensearch/cluster/metadata/IndexMetadata.java b/server/src/main/java/org/opensearch/cluster/metadata/IndexMetadata.java index 0161f4376c168..9adc052de0b48 100644 --- a/server/src/main/java/org/opensearch/cluster/metadata/IndexMetadata.java +++ b/server/src/main/java/org/opensearch/cluster/metadata/IndexMetadata.java @@ -59,6 +59,7 @@ import org.opensearch.common.settings.Setting; import org.opensearch.common.settings.Setting.Property; import org.opensearch.common.settings.Settings; +import org.opensearch.common.unit.TimeValue; import org.opensearch.common.xcontent.ToXContent; import org.opensearch.common.xcontent.ToXContentFragment; import org.opensearch.common.xcontent.XContentBuilder; @@ -301,6 +302,8 @@ public Iterator> settings() { public static final String SETTING_REMOTE_TRANSLOG_STORE_REPOSITORY = "index.remote_store.translog.repository"; + public static final String SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL = "index.remote_store.translog.buffer_interval"; + /** * Used to specify if the index data should be persisted in the remote store. */ @@ -446,6 +449,45 @@ public Iterator> settings() { Property.Final ); + public static final Setting INDEX_REMOTE_TRANSLOG_BUFFER_INTERVAL_SETTING = Setting.timeSetting( + SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL, + TimeValue.timeValueMillis(100), + TimeValue.timeValueMillis(50), + new Setting.Validator<>() { + + @Override + public void validate(final TimeValue value) {} + + @Override + public void validate(final TimeValue value, final Map, Object> settings) { + if (value == null) { + throw new IllegalArgumentException( + "Setting " + SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL + " should be provided with a valid time value" + ); + } else { + final Boolean isRemoteTranslogStoreEnabled = (Boolean) settings.get(INDEX_REMOTE_TRANSLOG_STORE_ENABLED_SETTING); + if (isRemoteTranslogStoreEnabled == null || isRemoteTranslogStoreEnabled == false) { + throw new IllegalArgumentException( + "Setting " + + SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL + + " can only be set when " + + SETTING_REMOTE_TRANSLOG_STORE_ENABLED + + " is set to true" + ); + } + } + } + + @Override + public Iterator> settings() { + final List> settings = Collections.singletonList(INDEX_REMOTE_TRANSLOG_STORE_ENABLED_SETTING); + return settings.iterator(); + } + }, + Property.IndexScope, + Property.Final + ); + public static final String SETTING_AUTO_EXPAND_REPLICAS = "index.auto_expand_replicas"; public static final Setting INDEX_AUTO_EXPAND_REPLICAS_SETTING = AutoExpandReplicas.SETTING; diff --git a/server/src/main/java/org/opensearch/common/settings/IndexScopedSettings.java b/server/src/main/java/org/opensearch/common/settings/IndexScopedSettings.java index 1efce2eba8867..a81c330177129 100644 --- a/server/src/main/java/org/opensearch/common/settings/IndexScopedSettings.java +++ b/server/src/main/java/org/opensearch/common/settings/IndexScopedSettings.java @@ -227,7 +227,8 @@ public final class IndexScopedSettings extends AbstractScopedSettings { IndexMetadata.INDEX_REMOTE_STORE_ENABLED_SETTING, IndexMetadata.INDEX_REMOTE_STORE_REPOSITORY_SETTING, IndexMetadata.INDEX_REMOTE_TRANSLOG_STORE_ENABLED_SETTING, - IndexMetadata.INDEX_REMOTE_TRANSLOG_REPOSITORY_SETTING + IndexMetadata.INDEX_REMOTE_TRANSLOG_REPOSITORY_SETTING, + IndexMetadata.INDEX_REMOTE_TRANSLOG_BUFFER_INTERVAL_SETTING ), FeatureFlags.SEARCHABLE_SNAPSHOT, List.of( diff --git a/server/src/main/java/org/opensearch/common/settings/Setting.java b/server/src/main/java/org/opensearch/common/settings/Setting.java index 588092976cc72..26a60e24b86b2 100644 --- a/server/src/main/java/org/opensearch/common/settings/Setting.java +++ b/server/src/main/java/org/opensearch/common/settings/Setting.java @@ -2173,6 +2173,23 @@ public static Setting timeSetting(String key, Setting fall return new Setting<>(key, fallbackSetting, (s) -> TimeValue.parseTimeValue(s, key), properties); } + public static Setting timeSetting( + String key, + TimeValue defaultValue, + TimeValue minValue, + Validator validator, + Property... properties + ) { + final SimpleKey simpleKey = new SimpleKey(key); + return new Setting<>( + simpleKey, + s -> defaultValue.getStringRep(), + minTimeValueParser(key, minValue, isFiltered(properties)), + validator, + properties + ); + } + public static Setting timeSetting( String key, Setting fallBackSetting, diff --git a/server/src/main/java/org/opensearch/common/util/concurrent/AsyncIOProcessor.java b/server/src/main/java/org/opensearch/common/util/concurrent/AsyncIOProcessor.java index 72cc0f5ee21d2..e9b9442c555e5 100644 --- a/server/src/main/java/org/opensearch/common/util/concurrent/AsyncIOProcessor.java +++ b/server/src/main/java/org/opensearch/common/util/concurrent/AsyncIOProcessor.java @@ -57,6 +57,7 @@ public abstract class AsyncIOProcessor { private final ArrayBlockingQueue>> queue; private final ThreadContext threadContext; private final Semaphore promiseSemaphore = new Semaphore(1); + private long lastRunStartTimeInNs; protected AsyncIOProcessor(Logger logger, int queueSize, ThreadContext threadContext) { this.logger = logger; @@ -67,7 +68,7 @@ protected AsyncIOProcessor(Logger logger, int queueSize, ThreadContext threadCon /** * Adds the given item to the queue. The listener is notified once the item is processed */ - public final void put(Item item, Consumer listener) { + public void put(Item item, Consumer listener) { Objects.requireNonNull(item, "item must not be null"); Objects.requireNonNull(listener, "listener must not be null"); // the algorithm here tires to reduce the load on each individual caller. @@ -78,12 +79,7 @@ public final void put(Item item, Consumer listener) { final boolean promised = promiseSemaphore.tryAcquire(); if (promised == false) { // in this case we are not responsible and can just block until there is space - try { - queue.put(new Tuple<>(item, preserveContext(listener))); - } catch (InterruptedException e) { - Thread.currentThread().interrupt(); - listener.accept(e); - } + addToQueue(item, listener); } // here we have to try to make the promise again otherwise there is a race when a thread puts an entry without making the promise @@ -104,7 +100,17 @@ public final void put(Item item, Consumer listener) { } } - private void drainAndProcessAndRelease(List>> candidates) { + void addToQueue(Item item, Consumer listener) { + try { + queue.put(new Tuple<>(item, preserveContext(listener))); + } catch (InterruptedException e) { + Thread.currentThread().interrupt(); + listener.accept(e); + } + } + + void drainAndProcessAndRelease(List>> candidates) { + lastRunStartTimeInNs = System.nanoTime(); Exception exception; try { queue.drainTo(candidates); @@ -130,7 +136,7 @@ private Exception processList(List>> candidates) return exception; } - private void notifyList(List>> candidates, Exception exception) { + void notifyList(List>> candidates, Exception exception) { for (Tuple> tuple : candidates) { Consumer consumer = tuple.v2(); try { @@ -141,7 +147,7 @@ private void notifyList(List>> candidates, Excep } } - private Consumer preserveContext(Consumer consumer) { + Consumer preserveContext(Consumer consumer) { Supplier restorableContext = threadContext.newRestorableContext(false); return e -> { try (ThreadContext.StoredContext ignore = restorableContext.get()) { @@ -154,4 +160,20 @@ private Consumer preserveContext(Consumer consumer) { * Writes or processes the items out or to disk. */ protected abstract void write(List>> candidates) throws IOException; + + Logger getLogger() { + return logger; + } + + Semaphore getPromiseSemaphore() { + return promiseSemaphore; + } + + long getLastRunStartTimeInNs() { + return lastRunStartTimeInNs; + } + + ArrayBlockingQueue>> getQueue() { + return queue; + } } diff --git a/server/src/main/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessor.java b/server/src/main/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessor.java new file mode 100644 index 0000000000000..e3157ea8dfdb4 --- /dev/null +++ b/server/src/main/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessor.java @@ -0,0 +1,93 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +package org.opensearch.common.util.concurrent; + +import org.apache.logging.log4j.Logger; +import org.opensearch.common.collect.Tuple; +import org.opensearch.common.unit.TimeValue; +import org.opensearch.threadpool.ThreadPool; + +import java.util.ArrayList; +import java.util.List; +import java.util.Objects; +import java.util.function.Consumer; + +/** + * A variant of {@link AsyncIOProcessor} that allows to batch and buffer processing items at every + * {@link BufferedAsyncIOProcessor#bufferInterval} in a separate threadpool. + *

+ * Requests are buffered till processor thread calls @{link drainAndProcessAndRelease} after bufferInterval. + * If more requests are enqueued between invocations of drainAndProcessAndRelease, another processor thread + * gets scheduled. Subsequent requests will get buffered till drainAndProcessAndRelease gets called in this new + * processor thread. + * + * @opensearch.internal + */ +public abstract class BufferedAsyncIOProcessor extends AsyncIOProcessor { + + private final ThreadPool threadpool; + private final TimeValue bufferInterval; + + protected BufferedAsyncIOProcessor( + Logger logger, + int queueSize, + ThreadContext threadContext, + ThreadPool threadpool, + TimeValue bufferInterval + ) { + super(logger, queueSize, threadContext); + this.threadpool = threadpool; + this.bufferInterval = bufferInterval; + } + + @Override + public void put(Item item, Consumer listener) { + Objects.requireNonNull(item, "item must not be null"); + Objects.requireNonNull(listener, "listener must not be null"); + addToQueue(item, listener); + scheduleProcess(); + } + + private void scheduleProcess() { + if (getQueue().isEmpty() == false && getPromiseSemaphore().tryAcquire()) { + try { + threadpool.schedule(this::process, getBufferInterval(), getBufferRefreshThreadPoolName()); + } catch (Exception e) { + getLogger().error("failed to schedule process"); + processSchedulingFailure(e); + getPromiseSemaphore().release(); + // This is to make sure that any new items that are added to the queue between processSchedulingFailure + // and releasing the semaphore is handled by a subsequent refresh and not starved. + scheduleProcess(); + } + } + } + + private void processSchedulingFailure(Exception e) { + List>> candidates = new ArrayList<>(); + getQueue().drainTo(candidates); + notifyList(candidates, e); + } + + private void process() { + drainAndProcessAndRelease(new ArrayList<>()); + scheduleProcess(); + } + + private TimeValue getBufferInterval() { + long timeSinceLastRunStartInNS = System.nanoTime() - getLastRunStartTimeInNs(); + if (timeSinceLastRunStartInNS >= bufferInterval.getNanos()) { + return TimeValue.ZERO; + } + return TimeValue.timeValueNanos(bufferInterval.getNanos() - timeSinceLastRunStartInNS); + } + + protected abstract String getBufferRefreshThreadPoolName(); + +} diff --git a/server/src/main/java/org/opensearch/index/IndexSettings.java b/server/src/main/java/org/opensearch/index/IndexSettings.java index dc54ace237070..de61ad7a6cfef 100644 --- a/server/src/main/java/org/opensearch/index/IndexSettings.java +++ b/server/src/main/java/org/opensearch/index/IndexSettings.java @@ -586,6 +586,7 @@ public final class IndexSettings { private final ReplicationType replicationType; private final boolean isRemoteStoreEnabled; private final boolean isRemoteTranslogStoreEnabled; + private final TimeValue remoteTranslogUploadBufferInterval; private final String remoteStoreTranslogRepository; private final String remoteStoreRepository; private final boolean isRemoteSnapshot; @@ -753,6 +754,10 @@ public IndexSettings(final IndexMetadata indexMetadata, final Settings nodeSetti isRemoteStoreEnabled = settings.getAsBoolean(IndexMetadata.SETTING_REMOTE_STORE_ENABLED, false); isRemoteTranslogStoreEnabled = settings.getAsBoolean(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_ENABLED, false); remoteStoreTranslogRepository = settings.get(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_REPOSITORY); + remoteTranslogUploadBufferInterval = settings.getAsTime( + IndexMetadata.SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL, + TimeValue.timeValueMillis(100) + ); remoteStoreRepository = settings.get(IndexMetadata.SETTING_REMOTE_STORE_REPOSITORY); isRemoteSnapshot = IndexModule.Type.REMOTE_SNAPSHOT.match(this.settings); @@ -1135,6 +1140,15 @@ public void setTranslogSyncInterval(TimeValue translogSyncInterval) { this.syncInterval = translogSyncInterval; } + /** + * Returns the translog sync/upload buffer interval when remote translog store is enabled and index setting + * {@code index.translog.durability} is set as {@code request}. + * @return the buffer interval. + */ + public TimeValue getRemoteTranslogUploadBufferInterval() { + return remoteTranslogUploadBufferInterval; + } + /** * Returns this interval in which the shards of this index are asynchronously refreshed. {@code -1} means async refresh is disabled. */ diff --git a/server/src/main/java/org/opensearch/index/shard/IndexShard.java b/server/src/main/java/org/opensearch/index/shard/IndexShard.java index 431519658edd1..fc37d1c5c7fe4 100644 --- a/server/src/main/java/org/opensearch/index/shard/IndexShard.java +++ b/server/src/main/java/org/opensearch/index/shard/IndexShard.java @@ -94,6 +94,7 @@ import org.opensearch.common.util.BigArrays; import org.opensearch.common.util.concurrent.AbstractRunnable; import org.opensearch.common.util.concurrent.AsyncIOProcessor; +import org.opensearch.common.util.concurrent.BufferedAsyncIOProcessor; import org.opensearch.common.util.concurrent.RunOnce; import org.opensearch.common.util.concurrent.ThreadContext; import org.opensearch.common.util.set.Sets; @@ -363,7 +364,13 @@ public IndexShard( this.indexSortSupplier = indexSortSupplier; this.indexEventListener = indexEventListener; this.threadPool = threadPool; - this.translogSyncProcessor = createTranslogSyncProcessor(logger, threadPool.getThreadContext(), this::getEngine); + this.translogSyncProcessor = createTranslogSyncProcessor( + logger, + threadPool, + this::getEngine, + indexSettings.isRemoteTranslogStoreEnabled(), + indexSettings.getRemoteTranslogUploadBufferInterval() + ); this.mapperService = mapperService; this.indexCache = indexCache; this.internalIndexingStats = new InternalIndexingStats(); @@ -3813,21 +3820,41 @@ public List getActiveOperations() { private static AsyncIOProcessor createTranslogSyncProcessor( Logger logger, - ThreadContext threadContext, - Supplier engineSupplier + ThreadPool threadPool, + Supplier engineSupplier, + boolean bufferAsyncIoProcessor, + TimeValue bufferInterval ) { - return new AsyncIOProcessor(logger, 1024, threadContext) { + ThreadContext threadContext = threadPool.getThreadContext(); + CheckedConsumer>>, IOException> writeConsumer = candidates -> { + try { + engineSupplier.get().translogManager().ensureTranslogSynced(candidates.stream().map(Tuple::v1)); + } catch (AlreadyClosedException ex) { + // that's fine since we already synced everything on engine close - this also is conform with the methods + // documentation + } catch (IOException ex) { // if this fails we are in deep shit - fail the request + logger.debug("failed to sync translog", ex); + throw ex; + } + }; + if (bufferAsyncIoProcessor) { + return new BufferedAsyncIOProcessor<>(logger, 102400, threadContext, threadPool, bufferInterval) { + @Override + protected void write(List>> candidates) throws IOException { + writeConsumer.accept(candidates); + } + + @Override + protected String getBufferRefreshThreadPoolName() { + return ThreadPool.Names.TRANSLOG_SYNC; + } + }; + } + + return new AsyncIOProcessor<>(logger, 1024, threadContext) { @Override protected void write(List>> candidates) throws IOException { - try { - engineSupplier.get().translogManager().ensureTranslogSynced(candidates.stream().map(Tuple::v1)); - } catch (AlreadyClosedException ex) { - // that's fine since we already synced everything on engine close - this also is conform with the methods - // documentation - } catch (IOException ex) { // if this fails we are in deep shit - fail the request - logger.debug("failed to sync translog", ex); - throw ex; - } + writeConsumer.accept(candidates); } }; } diff --git a/server/src/main/java/org/opensearch/threadpool/ThreadPool.java b/server/src/main/java/org/opensearch/threadpool/ThreadPool.java index 6f886de9ee88f..dd52e0f7ecf76 100644 --- a/server/src/main/java/org/opensearch/threadpool/ThreadPool.java +++ b/server/src/main/java/org/opensearch/threadpool/ThreadPool.java @@ -109,6 +109,7 @@ public static class Names { public static final String SYSTEM_READ = "system_read"; public static final String SYSTEM_WRITE = "system_write"; public static final String TRANSLOG_TRANSFER = "translog_transfer"; + public static final String TRANSLOG_SYNC = "translog_sync"; } /** @@ -174,6 +175,7 @@ public static ThreadPoolType fromType(String type) { map.put(Names.SYSTEM_READ, ThreadPoolType.FIXED); map.put(Names.SYSTEM_WRITE, ThreadPoolType.FIXED); map.put(Names.TRANSLOG_TRANSFER, ThreadPoolType.SCALING); + map.put(Names.TRANSLOG_SYNC, ThreadPoolType.FIXED); THREAD_POOL_TYPES = Collections.unmodifiableMap(map); } @@ -250,6 +252,7 @@ public ThreadPool( Names.TRANSLOG_TRANSFER, new ScalingExecutorBuilder(Names.TRANSLOG_TRANSFER, 1, halfProcMaxAt10, TimeValue.timeValueMinutes(5)) ); + builders.put(Names.TRANSLOG_SYNC, new FixedExecutorBuilder(settings, Names.TRANSLOG_SYNC, allocatedProcessors * 4, 10000)); for (final ExecutorBuilder builder : customBuilders) { if (builders.containsKey(builder.name())) { diff --git a/server/src/test/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessorTests.java b/server/src/test/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessorTests.java new file mode 100644 index 0000000000000..3ec7bdc54d3cb --- /dev/null +++ b/server/src/test/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessorTests.java @@ -0,0 +1,252 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +package org.opensearch.common.util.concurrent; + +import org.junit.After; +import org.junit.Before; +import org.opensearch.common.collect.Tuple; +import org.opensearch.common.settings.Settings; +import org.opensearch.common.unit.TimeValue; +import org.opensearch.test.OpenSearchTestCase; +import org.opensearch.threadpool.TestThreadPool; +import org.opensearch.threadpool.ThreadPool; + +import java.io.IOException; +import java.util.Collections; +import java.util.LinkedList; +import java.util.List; +import java.util.Map; +import java.util.concurrent.CountDownLatch; +import java.util.concurrent.Semaphore; +import java.util.concurrent.TimeUnit; +import java.util.concurrent.atomic.AtomicInteger; +import java.util.function.Consumer; +import java.util.stream.Collectors; +import java.util.stream.IntStream; + +import static java.lang.Thread.sleep; + +public class BufferedAsyncIOProcessorTests extends OpenSearchTestCase { + + private ThreadPool threadpool; + private ThreadContext threadContext; + + @Before + public void setUp() throws Exception { + super.setUp(); + threadpool = new TestThreadPool("buffered-async-io"); + threadContext = new ThreadContext(Settings.EMPTY); + } + + @After + public void cleanup() { + terminate(threadpool); + } + + public void testConsumerCanThrowExceptions() { + AtomicInteger received = new AtomicInteger(0); + AtomicInteger notified = new AtomicInteger(0); + + AsyncIOProcessor processor = new BufferedAsyncIOProcessor<>( + logger, + scaledRandomIntBetween(1, 2024), + threadContext, + threadpool, + TimeValue.timeValueMillis(50) + ) { + @Override + protected void write(List>> candidates) throws IOException { + received.addAndGet(candidates.size()); + } + + @Override + protected String getBufferRefreshThreadPoolName() { + return ThreadPool.Names.TRANSLOG_SYNC; + } + }; + processor.put(new Object(), (e) -> { + notified.incrementAndGet(); + throw new RuntimeException(); + }); + processor.put(new Object(), (e) -> { + notified.incrementAndGet(); + throw new RuntimeException(); + }); + try { + sleep(200); // Give the processor few chances to run + } catch (InterruptedException e) { + logger.error("Error while trying to sleep", e); + } + assertEquals(2, notified.get()); + assertEquals(2, received.get()); + } + + public void testPreserveThreadContext() throws InterruptedException { + final int threadCount = randomIntBetween(2, 10); + final String testHeader = "test-header"; + + AtomicInteger received = new AtomicInteger(0); + AtomicInteger notified = new AtomicInteger(0); + + CountDownLatch processed = new CountDownLatch(threadCount); + AsyncIOProcessor processor = new BufferedAsyncIOProcessor<>( + logger, + scaledRandomIntBetween(1, 2024), + threadContext, + threadpool, + TimeValue.timeValueMillis(100) + ) { + @Override + protected void write(List>> candidates) throws IOException { + received.addAndGet(candidates.size()); + } + + @Override + protected String getBufferRefreshThreadPoolName() { + return ThreadPool.Names.TRANSLOG_SYNC; + } + }; + + // all threads should be non-blocking. + List threads = IntStream.range(0, threadCount).mapToObj(i -> new Thread(getTestName() + "_" + i) { + private final String response = randomAlphaOfLength(10); + + { + setDaemon(true); + } + + @Override + public void run() { + threadContext.addResponseHeader(testHeader, response); + processor.put(new Object(), (e) -> { + final Map> expected = Collections.singletonMap(testHeader, Collections.singletonList(response)); + assertEquals(expected, threadContext.getResponseHeaders()); + notified.incrementAndGet(); + processed.countDown(); + }); + } + }).collect(Collectors.toList()); + threads.forEach(Thread::start); + threads.forEach(t -> { + try { + t.join(20000); + } catch (InterruptedException e) { + throw new RuntimeException(e); + } + }); + assertTrue(processed.await(1, TimeUnit.SECONDS)); + + assertEquals(threadCount, notified.get()); + assertEquals(threadCount, received.get()); + threads.forEach(t -> assertFalse(t.isAlive())); + } + + public void testSlowConsumer() { + AtomicInteger received = new AtomicInteger(0); + AtomicInteger notified = new AtomicInteger(0); + + AsyncIOProcessor processor = new BufferedAsyncIOProcessor<>( + logger, + scaledRandomIntBetween(1, 2024), + threadContext, + threadpool, + TimeValue.timeValueMillis(100) + ) { + @Override + protected void write(List>> candidates) throws IOException { + received.addAndGet(candidates.size()); + } + + @Override + protected String getBufferRefreshThreadPoolName() { + return ThreadPool.Names.TRANSLOG_SYNC; + } + }; + + int threadCount = randomIntBetween(2, 10); + Semaphore serializePutSemaphore = new Semaphore(1); + CountDownLatch allDone = new CountDownLatch(threadCount); + List threads = IntStream.range(0, threadCount).mapToObj(i -> new Thread(getTestName() + "_" + i) { + { + setDaemon(true); + } + + @Override + public void run() { + try { + assertTrue(serializePutSemaphore.tryAcquire(10, TimeUnit.SECONDS)); + } catch (InterruptedException e) { + throw new RuntimeException(e); + } + processor.put(new Object(), (e) -> { + serializePutSemaphore.release(); + notified.incrementAndGet(); + allDone.countDown(); + }); + } + }).collect(Collectors.toList()); + threads.forEach(Thread::start); + threads.forEach(t -> { + try { + t.join(20000); + } catch (InterruptedException e) { + throw new RuntimeException(e); + } + }); + try { + assertTrue(allDone.await(20000, TimeUnit.MILLISECONDS)); + } catch (InterruptedException e) { + throw new RuntimeException(e); + } + assertEquals(threadCount, notified.get()); + assertEquals(threadCount, received.get()); + threads.forEach(t -> assertFalse(t.isAlive())); + } + + public void testConsecutiveWritesAtLeastBufferIntervalAway() throws InterruptedException { + AtomicInteger received = new AtomicInteger(0); + AtomicInteger notified = new AtomicInteger(0); + long bufferIntervalMs = randomLongBetween(50, 150); + List writeInvocationTimes = new LinkedList<>(); + + AsyncIOProcessor processor = new BufferedAsyncIOProcessor<>( + logger, + scaledRandomIntBetween(1, 2024), + threadContext, + threadpool, + TimeValue.timeValueMillis(bufferIntervalMs) + ) { + @Override + protected void write(List>> candidates) throws IOException { + received.addAndGet(candidates.size()); + writeInvocationTimes.add(System.nanoTime()); + } + + @Override + protected String getBufferRefreshThreadPoolName() { + return ThreadPool.Names.TRANSLOG_SYNC; + } + }; + + int runCount = randomIntBetween(3, 10); + CountDownLatch processed = new CountDownLatch(runCount); + IntStream.range(0, runCount).forEach(i -> { + processor.put(new Object(), (e) -> { + notified.incrementAndGet(); + processed.countDown(); + }); + }); + assertTrue(processed.await(bufferIntervalMs * (runCount + 1), TimeUnit.MILLISECONDS)); + assertEquals(runCount, notified.get()); + assertEquals(runCount, received.get()); + for (int i = 1; i < writeInvocationTimes.size(); i++) { + assertTrue(writeInvocationTimes.get(i) - writeInvocationTimes.get(i - 1) >= bufferIntervalMs * 1_000_000); + } + } +} diff --git a/server/src/test/java/org/opensearch/index/IndexSettingsTests.java b/server/src/test/java/org/opensearch/index/IndexSettingsTests.java index 957eb337f5c85..28044410a21f4 100644 --- a/server/src/test/java/org/opensearch/index/IndexSettingsTests.java +++ b/server/src/test/java/org/opensearch/index/IndexSettingsTests.java @@ -964,11 +964,13 @@ public void testRemoteTranslogExplicitSetting() { .put(IndexMetadata.SETTING_VERSION_CREATED, Version.CURRENT) .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_ENABLED, true) .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_REPOSITORY, "tlog-store") + .put(IndexMetadata.INDEX_REMOTE_TRANSLOG_BUFFER_INTERVAL_SETTING.getKey(), "200ms") .build() ); IndexSettings settings = new IndexSettings(metadata, Settings.EMPTY); assertNull(settings.getRemoteStoreRepository()); assertEquals("tlog-store", settings.getRemoteStoreTranslogRepository()); + assertEquals(TimeValue.timeValueMillis(200), settings.getRemoteTranslogUploadBufferInterval()); } public void testSetRemoteTranslogRepositoryFailsWhenRemoteTranslogIsNotEnabled() { @@ -1000,6 +1002,47 @@ public void testSetRemoteTranslogRepositoryFailsWhenEmptyString() { assertEquals("Setting index.remote_store.translog.repository should be provided with non-empty repository ID", iae.getMessage()); } + public void testSetRemoteTranslogBufferIntervalDefaultSetting() { + Version createdVersion = VersionUtils.randomVersionBetween(random(), Version.V_2_0_0, Version.CURRENT); + Settings settings = Settings.builder() + .put(IndexMetadata.SETTING_INDEX_VERSION_CREATED.getKey(), createdVersion) + .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_ENABLED, true) + .build(); + assertEquals(TimeValue.timeValueMillis(100), IndexMetadata.INDEX_REMOTE_TRANSLOG_BUFFER_INTERVAL_SETTING.get(settings)); + } + + public void testSetRemoteTranslogBufferIntervalFailsWhenRemoteTranslogIsNotEnabled() { + Settings indexSettings = Settings.builder() + .put(IndexMetadata.SETTING_REPLICATION_TYPE, ReplicationType.SEGMENT) + .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_ENABLED, false) + .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL, "200ms") + .build(); + IllegalArgumentException iae = expectThrows( + IllegalArgumentException.class, + () -> IndexMetadata.INDEX_REMOTE_TRANSLOG_BUFFER_INTERVAL_SETTING.get(indexSettings) + ); + assertEquals( + "Setting index.remote_store.translog.buffer_interval can only be set when index.remote_store.translog.enabled is set to true", + iae.getMessage() + ); + } + + public void testSetRemoteTranslogBufferIntervalFailsWhenEmpty() { + Settings indexSettings = Settings.builder() + .put(IndexMetadata.SETTING_REPLICATION_TYPE, ReplicationType.SEGMENT) + .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_ENABLED, false) + .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL, "") + .build(); + IllegalArgumentException iae = expectThrows( + IllegalArgumentException.class, + () -> IndexMetadata.INDEX_REMOTE_TRANSLOG_BUFFER_INTERVAL_SETTING.get(indexSettings) + ); + assertEquals( + "failed to parse setting [index.remote_store.translog.buffer_interval] with value [] as a time value: unit is missing or unrecognized", + iae.getMessage() + ); + } + @SuppressForbidden(reason = "sets the SEARCHABLE_SNAPSHOT_EXTENDED_COMPATIBILITY feature flag") public void testExtendedCompatibilityVersionForRemoteSnapshot() throws Exception { try (FeatureFlagSetter f = FeatureFlagSetter.set(FeatureFlags.SEARCHABLE_SNAPSHOT_EXTENDED_COMPATIBILITY)) { diff --git a/server/src/test/java/org/opensearch/index/shard/ReplicaRecoveryWithRemoteTranslogOnPrimaryTests.java b/server/src/test/java/org/opensearch/index/shard/ReplicaRecoveryWithRemoteTranslogOnPrimaryTests.java index ba707cc30e6b8..d76afca51e354 100644 --- a/server/src/test/java/org/opensearch/index/shard/ReplicaRecoveryWithRemoteTranslogOnPrimaryTests.java +++ b/server/src/test/java/org/opensearch/index/shard/ReplicaRecoveryWithRemoteTranslogOnPrimaryTests.java @@ -36,6 +36,7 @@ public class ReplicaRecoveryWithRemoteTranslogOnPrimaryTests extends OpenSearchI .put(IndexMetadata.SETTING_REMOTE_STORE_ENABLED, "true") .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_ENABLED, "true") .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_REPOSITORY, "translog-repo") + .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL, "100ms") .build(); public void testStartSequenceForReplicaRecovery() throws Exception { diff --git a/server/src/test/java/org/opensearch/indices/recovery/RemoteStorePeerRecoverySourceHandlerTests.java b/server/src/test/java/org/opensearch/indices/recovery/RemoteStorePeerRecoverySourceHandlerTests.java index 465629406b54b..33c748f46bd86 100644 --- a/server/src/test/java/org/opensearch/indices/recovery/RemoteStorePeerRecoverySourceHandlerTests.java +++ b/server/src/test/java/org/opensearch/indices/recovery/RemoteStorePeerRecoverySourceHandlerTests.java @@ -23,6 +23,7 @@ public class RemoteStorePeerRecoverySourceHandlerTests extends OpenSearchIndexLe .put(IndexMetadata.SETTING_REMOTE_STORE_ENABLED, "true") .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_ENABLED, "true") .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_STORE_REPOSITORY, "translog-repo") + .put(IndexMetadata.SETTING_REMOTE_TRANSLOG_BUFFER_INTERVAL, "100ms") .build(); public void testReplicaShardRecoveryUptoLastFlushedCommit() throws Exception { diff --git a/server/src/test/java/org/opensearch/threadpool/ScalingThreadPoolTests.java b/server/src/test/java/org/opensearch/threadpool/ScalingThreadPoolTests.java index cd941feb37002..eca5a8eb19e47 100644 --- a/server/src/test/java/org/opensearch/threadpool/ScalingThreadPoolTests.java +++ b/server/src/test/java/org/opensearch/threadpool/ScalingThreadPoolTests.java @@ -133,6 +133,7 @@ private int expectedSize(final String threadPoolName, final int numberOfProcesso sizes.put(ThreadPool.Names.FETCH_SHARD_STARTED, ThreadPool::twiceAllocatedProcessors); sizes.put(ThreadPool.Names.FETCH_SHARD_STORE, ThreadPool::twiceAllocatedProcessors); sizes.put(ThreadPool.Names.TRANSLOG_TRANSFER, ThreadPool::halfAllocatedProcessorsMaxTen); + sizes.put(ThreadPool.Names.TRANSLOG_SYNC, n -> 4 * n); return sizes.get(threadPoolName).apply(numberOfProcessors); } From 249f1a6995a4318177e36e39c5eac28ae93ae022 Mon Sep 17 00:00:00 2001 From: Rishab Nahata Date: Sun, 29 Jan 2023 21:08:38 +0530 Subject: [PATCH 03/20] Cluster health call to throw decommissioned exception for local decommissioned node (#6008) * Cluster health call to throw decommissioned exception for local decommissioned nodes Signed-off-by: Rishab Nahata --- CHANGELOG.md | 1 + .../rest-api-spec/api/cluster.health.json | 4 +++ .../AwarenessAttributeDecommissionIT.java | 30 +++++++++++++++++++ .../cluster/health/ClusterHealthRequest.java | 24 +++++++++++++++ .../health/ClusterHealthRequestBuilder.java | 8 +++++ .../health/TransportClusterHealthAction.java | 15 ++++++++-- .../cluster/coordination/Coordinator.java | 3 +- .../NodeDecommissionedException.java | 6 ++++ .../cluster/RestClusterHealthAction.java | 3 ++ .../health/ClusterHealthRequestTests.java | 18 +++++++++++ .../health/ClusterStateHealthTests.java | 3 +- .../cluster/RestClusterHealthActionTests.java | 6 ++++ 12 files changed, 116 insertions(+), 5 deletions(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index c3beb5a2d3f20..6165beed00b0e 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -88,6 +88,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), ### Changed - Use ReplicationFailedException instead of OpensearchException in ReplicationTarget ([#4725](https://github.com/opensearch-project/OpenSearch/pull/4725)) - [Refactor] Use local opensearch.common.SetOnce instead of lucene's utility class ([#5947](https://github.com/opensearch-project/OpenSearch/pull/5947)) +- Cluster health call to throw decommissioned exception for local decommissioned node([#6008](https://github.com/opensearch-project/OpenSearch/pull/6008)) ### Deprecated diff --git a/rest-api-spec/src/main/resources/rest-api-spec/api/cluster.health.json b/rest-api-spec/src/main/resources/rest-api-spec/api/cluster.health.json index 2fc9a2d4fd716..a7c5dac5fe414 100644 --- a/rest-api-spec/src/main/resources/rest-api-spec/api/cluster.health.json +++ b/rest-api-spec/src/main/resources/rest-api-spec/api/cluster.health.json @@ -111,6 +111,10 @@ "awareness_attribute":{ "type":"string", "description":"The awareness attribute for which the health is required" + }, + "ensure_node_commissioned":{ + "type":"boolean", + "description": "Checks whether local node is commissioned or not. If set to true on a local call it will throw exception if node is decommissioned (default: false)" } } } diff --git a/server/src/internalClusterTest/java/org/opensearch/cluster/coordination/AwarenessAttributeDecommissionIT.java b/server/src/internalClusterTest/java/org/opensearch/cluster/coordination/AwarenessAttributeDecommissionIT.java index e2eb08bd0969c..676cebce9e6af 100644 --- a/server/src/internalClusterTest/java/org/opensearch/cluster/coordination/AwarenessAttributeDecommissionIT.java +++ b/server/src/internalClusterTest/java/org/opensearch/cluster/coordination/AwarenessAttributeDecommissionIT.java @@ -305,6 +305,36 @@ public boolean innerMatch(LogEvent event) { Coordinator coordinator = (Coordinator) internalCluster().getInstance(Discovery.class, decommissionedNode); assertFalse(coordinator.localNodeCommissioned()); + // Check cluster health API for decommissioned and active node + ClusterHealthResponse activeNodeLocalHealth = client(activeNode).admin() + .cluster() + .prepareHealth() + .setLocal(true) + .setEnsureNodeCommissioned(true) + .execute() + .actionGet(); + assertFalse(activeNodeLocalHealth.isTimedOut()); + + ClusterHealthResponse decommissionedNodeLocalHealth = client(decommissionedNode).admin() + .cluster() + .prepareHealth() + .setLocal(true) + .execute() + .actionGet(); + assertFalse(decommissionedNodeLocalHealth.isTimedOut()); + + NodeDecommissionedException ex = expectThrows( + NodeDecommissionedException.class, + () -> client(decommissionedNode).admin() + .cluster() + .prepareHealth() + .setLocal(true) + .setEnsureNodeCommissioned(true) + .execute() + .actionGet() + ); + assertTrue(ex.getMessage().contains("local node is decommissioned")); + // Recommissioning the zone back to gracefully succeed the test once above tests succeeds DeleteDecommissionStateResponse deleteDecommissionStateResponse = client(activeNode).execute( DeleteDecommissionStateAction.INSTANCE, diff --git a/server/src/main/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequest.java b/server/src/main/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequest.java index e4c07dc1e045d..3a1288cef0ea5 100644 --- a/server/src/main/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequest.java +++ b/server/src/main/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequest.java @@ -67,6 +67,7 @@ public class ClusterHealthRequest extends ClusterManagerNodeReadRequesttrue if local information is to be returned only when local node is also commissioned + * false to not check local node if commissioned or not for a local request + */ + public final boolean ensureNodeCommissioned() { + return ensureNodeCommissioned; + } + @Override public ActionRequestValidationException validate() { if (level.equals(Level.AWARENESS_ATTRIBUTES) && indices.length > 0) { @@ -321,6 +342,9 @@ public ActionRequestValidationException validate() { } else if (!level.equals(Level.AWARENESS_ATTRIBUTES) && awarenessAttribute != null) { return addValidationError("level=awareness_attributes is required with awareness_attribute parameter", null); } + if (ensureNodeCommissioned && local == false) { + return addValidationError("not a local request to ensure local node commissioned", null); + } return null; } diff --git a/server/src/main/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequestBuilder.java b/server/src/main/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequestBuilder.java index ac9697e06a5d7..98d19b8e32247 100644 --- a/server/src/main/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequestBuilder.java +++ b/server/src/main/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequestBuilder.java @@ -161,4 +161,12 @@ public ClusterHealthRequestBuilder setLevel(String level) { request.setLevel(level); return this; } + + /** + * Specifies if the local request should ensure that the local node is commissioned + */ + public final ClusterHealthRequestBuilder setEnsureNodeCommissioned(boolean ensureNodeCommissioned) { + request.ensureNodeCommissioned(ensureNodeCommissioned); + return this; + } } diff --git a/server/src/main/java/org/opensearch/action/admin/cluster/health/TransportClusterHealthAction.java b/server/src/main/java/org/opensearch/action/admin/cluster/health/TransportClusterHealthAction.java index a0b760d0dce28..a94631aae066f 100644 --- a/server/src/main/java/org/opensearch/action/admin/cluster/health/TransportClusterHealthAction.java +++ b/server/src/main/java/org/opensearch/action/admin/cluster/health/TransportClusterHealthAction.java @@ -46,6 +46,8 @@ import org.opensearch.cluster.LocalClusterUpdateTask; import org.opensearch.cluster.NotClusterManagerException; import org.opensearch.cluster.block.ClusterBlockException; +import org.opensearch.cluster.coordination.Coordinator; +import org.opensearch.cluster.decommission.NodeDecommissionedException; import org.opensearch.cluster.health.ClusterHealthStatus; import org.opensearch.cluster.metadata.IndexNameExpressionResolver; import org.opensearch.cluster.metadata.ProcessClusterEventTimeoutException; @@ -57,6 +59,7 @@ import org.opensearch.common.io.stream.StreamInput; import org.opensearch.common.unit.TimeValue; import org.opensearch.common.util.CollectionUtils; +import org.opensearch.discovery.Discovery; import org.opensearch.index.IndexNotFoundException; import org.opensearch.node.NodeClosedException; import org.opensearch.tasks.Task; @@ -77,6 +80,7 @@ public class TransportClusterHealthAction extends TransportClusterManagerNodeRea private static final Logger logger = LogManager.getLogger(TransportClusterHealthAction.class); private final AllocationService allocationService; + private final Discovery discovery; @Inject public TransportClusterHealthAction( @@ -85,7 +89,8 @@ public TransportClusterHealthAction( ThreadPool threadPool, ActionFilters actionFilters, IndexNameExpressionResolver indexNameExpressionResolver, - AllocationService allocationService + AllocationService allocationService, + Discovery discovery ) { super( ClusterHealthAction.NAME, @@ -98,6 +103,7 @@ public TransportClusterHealthAction( indexNameExpressionResolver ); this.allocationService = allocationService; + this.discovery = discovery; } @Override @@ -134,7 +140,12 @@ protected void clusterManagerOperation( final ClusterState unusedState, final ActionListener listener ) { - + if (request.ensureNodeCommissioned() + && discovery instanceof Coordinator + && ((Coordinator) discovery).localNodeCommissioned() == false) { + listener.onFailure(new NodeDecommissionedException("local node is decommissioned")); + return; + } final int waitCount = getWaitCount(request); if (request.waitForEvents() != null) { diff --git a/server/src/main/java/org/opensearch/cluster/coordination/Coordinator.java b/server/src/main/java/org/opensearch/cluster/coordination/Coordinator.java index b4c9380a76b10..dd7363426af19 100644 --- a/server/src/main/java/org/opensearch/cluster/coordination/Coordinator.java +++ b/server/src/main/java/org/opensearch/cluster/coordination/Coordinator.java @@ -1444,8 +1444,7 @@ synchronized void onNodeCommissionStatusChange(boolean localNodeCommissioned) { peerFinder.onNodeCommissionStatusChange(localNodeCommissioned); } - // package-visible for testing - boolean localNodeCommissioned() { + public boolean localNodeCommissioned() { return localNodeCommissioned; } diff --git a/server/src/main/java/org/opensearch/cluster/decommission/NodeDecommissionedException.java b/server/src/main/java/org/opensearch/cluster/decommission/NodeDecommissionedException.java index 847d5a527b017..c91509a0db161 100644 --- a/server/src/main/java/org/opensearch/cluster/decommission/NodeDecommissionedException.java +++ b/server/src/main/java/org/opensearch/cluster/decommission/NodeDecommissionedException.java @@ -10,6 +10,7 @@ import org.opensearch.OpenSearchException; import org.opensearch.common.io.stream.StreamInput; +import org.opensearch.rest.RestStatus; import java.io.IOException; @@ -28,4 +29,9 @@ public NodeDecommissionedException(String msg, Object... args) { public NodeDecommissionedException(StreamInput in) throws IOException { super(in); } + + @Override + public RestStatus status() { + return RestStatus.FAILED_DEPENDENCY; + } } diff --git a/server/src/main/java/org/opensearch/rest/action/admin/cluster/RestClusterHealthAction.java b/server/src/main/java/org/opensearch/rest/action/admin/cluster/RestClusterHealthAction.java index d8a629821954b..04b82a536a4a9 100644 --- a/server/src/main/java/org/opensearch/rest/action/admin/cluster/RestClusterHealthAction.java +++ b/server/src/main/java/org/opensearch/rest/action/admin/cluster/RestClusterHealthAction.java @@ -89,6 +89,9 @@ public static ClusterHealthRequest fromRequest(final RestRequest request) { final ClusterHealthRequest clusterHealthRequest = clusterHealthRequest(Strings.splitStringByCommaToArray(request.param("index"))); clusterHealthRequest.indicesOptions(IndicesOptions.fromRequest(request, clusterHealthRequest.indicesOptions())); clusterHealthRequest.local(request.paramAsBoolean("local", clusterHealthRequest.local())); + clusterHealthRequest.ensureNodeCommissioned( + request.paramAsBoolean("ensure_node_commissioned", clusterHealthRequest.ensureNodeCommissioned()) + ); clusterHealthRequest.clusterManagerNodeTimeout( request.paramAsTime("cluster_manager_timeout", clusterHealthRequest.clusterManagerNodeTimeout()) ); diff --git a/server/src/test/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequestTests.java b/server/src/test/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequestTests.java index 2576823578630..b87194ac717c5 100644 --- a/server/src/test/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequestTests.java +++ b/server/src/test/java/org/opensearch/action/admin/cluster/health/ClusterHealthRequestTests.java @@ -32,6 +32,7 @@ package org.opensearch.action.admin.cluster.health; +import org.opensearch.action.ActionRequestValidationException; import org.opensearch.action.support.IndicesOptions; import org.opensearch.cluster.health.ClusterHealthStatus; import org.opensearch.common.Priority; @@ -70,6 +71,23 @@ public void testRequestReturnsHiddenIndicesByDefault() { assertTrue(defaultRequest.indicesOptions().expandWildcardsHidden()); } + public void testValidation() { + ClusterHealthRequest clusterHealthRequest = randomRequest(); + { + clusterHealthRequest.local(false); + clusterHealthRequest.ensureNodeCommissioned(true); + ActionRequestValidationException e = clusterHealthRequest.validate(); + assertNotNull(e); + assertTrue(e.getMessage().contains("not a local request to ensure local node commissioned")); + } + { + clusterHealthRequest.local(true); + clusterHealthRequest.ensureNodeCommissioned(false); + ActionRequestValidationException e = clusterHealthRequest.validate(); + assertNull(e); + } + } + private ClusterHealthRequest randomRequest() { ClusterHealthRequest request = new ClusterHealthRequest(); request.waitForStatus(randomFrom(ClusterHealthStatus.values())); diff --git a/server/src/test/java/org/opensearch/cluster/health/ClusterStateHealthTests.java b/server/src/test/java/org/opensearch/cluster/health/ClusterStateHealthTests.java index 80e76d44b9c41..19458c9f8a4f3 100644 --- a/server/src/test/java/org/opensearch/cluster/health/ClusterStateHealthTests.java +++ b/server/src/test/java/org/opensearch/cluster/health/ClusterStateHealthTests.java @@ -177,7 +177,8 @@ public void testClusterHealthWaitsForClusterStateApplication() throws Interrupte threadPool, new ActionFilters(new HashSet<>()), indexNameExpressionResolver, - new AllocationService(null, new TestGatewayAllocator(), null, null, null) + new AllocationService(null, new TestGatewayAllocator(), null, null, null), + null ); PlainActionFuture listener = new PlainActionFuture<>(); action.execute(new ClusterHealthRequest().waitForGreenStatus(), listener); diff --git a/server/src/test/java/org/opensearch/rest/action/admin/cluster/RestClusterHealthActionTests.java b/server/src/test/java/org/opensearch/rest/action/admin/cluster/RestClusterHealthActionTests.java index 975a4d8120965..4c60ea810f591 100644 --- a/server/src/test/java/org/opensearch/rest/action/admin/cluster/RestClusterHealthActionTests.java +++ b/server/src/test/java/org/opensearch/rest/action/admin/cluster/RestClusterHealthActionTests.java @@ -52,6 +52,10 @@ public void testFromRequest() { Map params = new HashMap<>(); String index = "index"; boolean local = randomBoolean(); + boolean ensureLocalNodeCommissioned = false; + if (local) { + ensureLocalNodeCommissioned = randomBoolean(); + } String clusterManagerTimeout = randomTimeValue(); String timeout = randomTimeValue(); ClusterHealthStatus waitForStatus = randomFrom(ClusterHealthStatus.values()); @@ -63,6 +67,7 @@ public void testFromRequest() { params.put("index", index); params.put("local", String.valueOf(local)); + params.put("ensure_node_commissioned", String.valueOf(ensureLocalNodeCommissioned)); params.put("cluster_manager_timeout", clusterManagerTimeout); params.put("timeout", timeout); params.put("wait_for_status", waitForStatus.name()); @@ -81,6 +86,7 @@ public void testFromRequest() { assertThat(clusterHealthRequest.indices().length, equalTo(1)); assertThat(clusterHealthRequest.indices()[0], equalTo(index)); assertThat(clusterHealthRequest.local(), equalTo(local)); + assertThat(clusterHealthRequest.ensureNodeCommissioned(), equalTo(ensureLocalNodeCommissioned)); assertThat(clusterHealthRequest.clusterManagerNodeTimeout(), equalTo(TimeValue.parseTimeValue(clusterManagerTimeout, "test"))); assertThat(clusterHealthRequest.timeout(), equalTo(TimeValue.parseTimeValue(timeout, "test"))); assertThat(clusterHealthRequest.waitForStatus(), equalTo(waitForStatus)); From cd860ec9e4918464433a0a3030fe330359e8fdfe Mon Sep 17 00:00:00 2001 From: Anshu Agarwal Date: Sun, 29 Jan 2023 22:31:24 +0530 Subject: [PATCH 04/20] Fix weighted shard routing state across search requests (#6004) * Fix maintaining state across search requests with weighted shard routing Signed-off-by: Anshu Agarwal --- CHANGELOG.md | 1 + .../routing/IndexShardRoutingTable.java | 7 +- .../structure/RoutingIteratorTests.java | 92 +++++++++++++++++++ 3 files changed, 98 insertions(+), 2 deletions(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index 6165beed00b0e..bc1caf276d6ca 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -96,6 +96,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), ### Fixed - [Segment Replication] Fix for peer recovery ([#5344](https://github.com/opensearch-project/OpenSearch/pull/5344)) +- Fix weighted shard routing state across search requests([#6004](https://github.com/opensearch-project/OpenSearch/pull/6004)) ### Security diff --git a/server/src/main/java/org/opensearch/cluster/routing/IndexShardRoutingTable.java b/server/src/main/java/org/opensearch/cluster/routing/IndexShardRoutingTable.java index 207570c1d56b2..f730a2833fd02 100644 --- a/server/src/main/java/org/opensearch/cluster/routing/IndexShardRoutingTable.java +++ b/server/src/main/java/org/opensearch/cluster/routing/IndexShardRoutingTable.java @@ -77,6 +77,8 @@ public class IndexShardRoutingTable implements Iterable { final ShardShuffler shuffler; + // Shuffler for weighted round-robin shard routing. This uses rotation to permute shards. + final ShardShuffler shufflerForWeightedRouting; final ShardId shardId; final ShardRouting primary; @@ -105,6 +107,7 @@ public class IndexShardRoutingTable implements Iterable { IndexShardRoutingTable(ShardId shardId, List shards) { this.shardId = shardId; this.shuffler = new RotationShardShuffler(Randomness.get().nextInt()); + this.shufflerForWeightedRouting = new RotationShardShuffler(Randomness.get().nextInt()); this.shards = Collections.unmodifiableList(shards); ShardRouting primary = null; @@ -323,11 +326,11 @@ public ShardIterator activeInitializingShardsWeightedIt( double defaultWeight, boolean isFailOpenEnabled ) { - final int seed = shuffler.nextSeed(); + final int seed = shufflerForWeightedRouting.nextSeed(); List ordered = new ArrayList<>(); List orderedActiveShards = getActiveShardsByWeight(weightedRouting, nodes, defaultWeight); List orderedListWithDistinctShards; - ordered.addAll(shuffler.shuffle(orderedActiveShards, seed)); + ordered.addAll(shufflerForWeightedRouting.shuffle(orderedActiveShards, seed)); if (!allInitializingShards.isEmpty()) { List orderedInitializingShards = getInitializingShardsByWeight(weightedRouting, nodes, defaultWeight); ordered.addAll(orderedInitializingShards); diff --git a/server/src/test/java/org/opensearch/cluster/structure/RoutingIteratorTests.java b/server/src/test/java/org/opensearch/cluster/structure/RoutingIteratorTests.java index 4196b8882fa8f..9715d3af09fc7 100644 --- a/server/src/test/java/org/opensearch/cluster/structure/RoutingIteratorTests.java +++ b/server/src/test/java/org/opensearch/cluster/structure/RoutingIteratorTests.java @@ -59,10 +59,12 @@ import org.opensearch.test.ClusterServiceUtils; import org.opensearch.threadpool.TestThreadPool; +import java.util.ArrayList; import java.util.Arrays; import java.util.Collections; import java.util.HashMap; import java.util.Iterator; +import java.util.List; import java.util.Map; import static java.util.Collections.singletonMap; @@ -768,4 +770,94 @@ public void testWeightedRoutingShardState() { terminate(threadPool); } } + + /** + * Test to validate that shard routing state is maintained across requests, requests are assigned to nodes + * according to assigned routing weights + */ + public void testWeightedRoutingShardStateWithDifferentWeights() { + TestThreadPool threadPool = null; + try { + Settings.Builder settings = Settings.builder() + .put("cluster.routing.allocation.node_concurrent_recoveries", 10) + .put("cluster.routing.allocation.awareness.attributes", "zone"); + AllocationService strategy = createAllocationService(settings.build()); + + Metadata metadata = Metadata.builder() + .put(IndexMetadata.builder("test").settings(settings(Version.CURRENT)).numberOfShards(1).numberOfReplicas(2)) + .build(); + + RoutingTable routingTable = RoutingTable.builder().addAsNew(metadata.index("test")).build(); + + ClusterState clusterState = ClusterState.builder(ClusterName.CLUSTER_NAME_SETTING.getDefault(Settings.EMPTY)) + .metadata(metadata) + .routingTable(routingTable) + .build(); + + threadPool = new TestThreadPool("testThatOnlyNodesSupport"); + ClusterService clusterService = ClusterServiceUtils.createClusterService(threadPool); + + Map node1Attributes = new HashMap<>(); + node1Attributes.put("zone", "zone1"); + Map node2Attributes = new HashMap<>(); + node2Attributes.put("zone", "zone2"); + Map node3Attributes = new HashMap<>(); + node3Attributes.put("zone", "zone3"); + clusterState = ClusterState.builder(clusterState) + .nodes( + DiscoveryNodes.builder() + .add(newNode("node1", unmodifiableMap(node1Attributes))) + .add(newNode("node2", unmodifiableMap(node2Attributes))) + .add(newNode("node3", unmodifiableMap(node3Attributes))) + .localNodeId("node1") + ) + .build(); + clusterState = strategy.reroute(clusterState, "reroute"); + + clusterState = startInitializingShardsAndReroute(strategy, clusterState); + clusterState = startInitializingShardsAndReroute(strategy, clusterState); + List> weightsList = new ArrayList<>(); + Map weights1 = Map.of("zone1", 1.0, "zone2", 1.0, "zone3", 0.0); + weightsList.add(weights1); + + Map weights2 = Map.of("zone1", 1.0, "zone2", 0.0, "zone3", 1.0); + weightsList.add(weights2); + + Map weights3 = Map.of("zone1", 0.0, "zone2", 1.0, "zone3", 1.0); + weightsList.add(weights3); + + Map weights4 = Map.of("zone1", 1.0, "zone2", 1.0, "zone3", 0.0); + weightsList.add(weights4); + + for (int i = 0; i < weightsList.size(); i++) { + WeightedRouting weightedRouting = new WeightedRouting("zone", weightsList.get(i)); + ShardIterator shardIterator = clusterState.routingTable() + .index("test") + .shard(0) + .activeInitializingShardsWeightedIt(weightedRouting, clusterState.nodes(), 1, true); + + ShardRouting shardRouting1 = shardIterator.nextOrNull(); + + shardIterator = clusterState.routingTable() + .index("test") + .shard(0) + .activeInitializingShardsWeightedIt(weightedRouting, clusterState.nodes(), 1, true); + + ShardRouting shardRouting2 = shardIterator.nextOrNull(); + + shardIterator = clusterState.routingTable() + .index("test") + .shard(0) + .activeInitializingShardsWeightedIt(weightedRouting, clusterState.nodes(), 1, true); + + ShardRouting shardRouting3 = shardIterator.nextOrNull(); + + assertEquals(shardRouting1.currentNodeId(), shardRouting3.currentNodeId()); + assertNotEquals(shardRouting1.currentNodeId(), shardRouting2.currentNodeId()); + } + + } finally { + terminate(threadPool); + } + } } From 261814294b449e48d747f35688e4aaaf515784fe Mon Sep 17 00:00:00 2001 From: Ashish Date: Mon, 30 Jan 2023 12:32:33 +0530 Subject: [PATCH 05/20] Refactor buffer interval code for remote-backed indexes (#6063) Signed-off-by: Ashish Singh --- .../common/util/concurrent/BufferedAsyncIOProcessor.java | 4 ++-- .../main/java/org/opensearch/index/shard/IndexShard.java | 2 +- .../util/concurrent/BufferedAsyncIOProcessorTests.java | 8 ++++---- 3 files changed, 7 insertions(+), 7 deletions(-) diff --git a/server/src/main/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessor.java b/server/src/main/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessor.java index e3157ea8dfdb4..f3d909e1c92bc 100644 --- a/server/src/main/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessor.java +++ b/server/src/main/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessor.java @@ -57,7 +57,7 @@ public void put(Item item, Consumer listener) { private void scheduleProcess() { if (getQueue().isEmpty() == false && getPromiseSemaphore().tryAcquire()) { try { - threadpool.schedule(this::process, getBufferInterval(), getBufferRefreshThreadPoolName()); + threadpool.schedule(this::process, getBufferInterval(), getBufferProcessThreadPoolName()); } catch (Exception e) { getLogger().error("failed to schedule process"); processSchedulingFailure(e); @@ -88,6 +88,6 @@ private TimeValue getBufferInterval() { return TimeValue.timeValueNanos(bufferInterval.getNanos() - timeSinceLastRunStartInNS); } - protected abstract String getBufferRefreshThreadPoolName(); + protected abstract String getBufferProcessThreadPoolName(); } diff --git a/server/src/main/java/org/opensearch/index/shard/IndexShard.java b/server/src/main/java/org/opensearch/index/shard/IndexShard.java index fc37d1c5c7fe4..3b098bc1fc93c 100644 --- a/server/src/main/java/org/opensearch/index/shard/IndexShard.java +++ b/server/src/main/java/org/opensearch/index/shard/IndexShard.java @@ -3845,7 +3845,7 @@ protected void write(List>> candida } @Override - protected String getBufferRefreshThreadPoolName() { + protected String getBufferProcessThreadPoolName() { return ThreadPool.Names.TRANSLOG_SYNC; } }; diff --git a/server/src/test/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessorTests.java b/server/src/test/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessorTests.java index 3ec7bdc54d3cb..85bd715f73aba 100644 --- a/server/src/test/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessorTests.java +++ b/server/src/test/java/org/opensearch/common/util/concurrent/BufferedAsyncIOProcessorTests.java @@ -66,7 +66,7 @@ protected void write(List>> candidates) throws } @Override - protected String getBufferRefreshThreadPoolName() { + protected String getBufferProcessThreadPoolName() { return ThreadPool.Names.TRANSLOG_SYNC; } }; @@ -108,7 +108,7 @@ protected void write(List>> candidates) throws } @Override - protected String getBufferRefreshThreadPoolName() { + protected String getBufferProcessThreadPoolName() { return ThreadPool.Names.TRANSLOG_SYNC; } }; @@ -164,7 +164,7 @@ protected void write(List>> candidates) throws } @Override - protected String getBufferRefreshThreadPoolName() { + protected String getBufferProcessThreadPoolName() { return ThreadPool.Names.TRANSLOG_SYNC; } }; @@ -229,7 +229,7 @@ protected void write(List>> candidates) throws } @Override - protected String getBufferRefreshThreadPoolName() { + protected String getBufferProcessThreadPoolName() { return ThreadPool.Names.TRANSLOG_SYNC; } }; From 01d52b019e76b2c760c18a3271a64781742f3e9b Mon Sep 17 00:00:00 2001 From: "dependabot[bot]" <49699333+dependabot[bot]@users.noreply.github.com> Date: Mon, 30 Jan 2023 11:28:47 -0500 Subject: [PATCH 06/20] Bump jna from 5.11.0 to 5.13.0 in /buildSrc (#6071) * Bump jna from 5.11.0 to 5.13.0 in /buildSrc Bumps [jna](https://github.com/java-native-access/jna) from 5.11.0 to 5.13.0. - [Release notes](https://github.com/java-native-access/jna/releases) - [Changelog](https://github.com/java-native-access/jna/blob/master/CHANGES.md) - [Commits](https://github.com/java-native-access/jna/compare/5.11.0...5.13.0) --- updated-dependencies: - dependency-name: net.java.dev.jna:jna dependency-type: direct:production update-type: version-update:semver-minor ... Signed-off-by: dependabot[bot] * Update changelog Signed-off-by: dependabot[bot] --------- Signed-off-by: dependabot[bot] Co-authored-by: dependabot[bot] <49699333+dependabot[bot]@users.noreply.github.com> Co-authored-by: dependabot[bot] --- CHANGELOG.md | 3 ++- buildSrc/build.gradle | 2 +- 2 files changed, 3 insertions(+), 2 deletions(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index bc1caf276d6ca..28d3fd9beb0f8 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -37,6 +37,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Bumps `wiremock-jre8-standalone` from 2.33.2 to 2.35.0 - Bumps `gson` from 2.10 to 2.10.1 - Bumps `json-schema-validator` from 1.0.73 to 1.0.76 +- Bumps `jna` from 5.11.0 to 5.13.0 ### Changed - [CCR] Add getHistoryOperationsFromTranslog method to fetch the history snapshot from translogs ([#3948](https://github.com/opensearch-project/OpenSearch/pull/3948)) @@ -101,4 +102,4 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), ### Security [Unreleased 3.0]: https://github.com/opensearch-project/OpenSearch/compare/2.x...HEAD -[Unreleased 2.x]: https://github.com/opensearch-project/OpenSearch/compare/2.5...2.x +[Unreleased 2.x]: https://github.com/opensearch-project/OpenSearch/compare/2.5...2.x \ No newline at end of file diff --git a/buildSrc/build.gradle b/buildSrc/build.gradle index 58849bb637429..070c4493ee687 100644 --- a/buildSrc/build.gradle +++ b/buildSrc/build.gradle @@ -110,7 +110,7 @@ dependencies { api 'com.netflix.nebula:gradle-info-plugin:12.0.0' api 'org.apache.rat:apache-rat:0.15' api 'commons-io:commons-io:2.7' - api "net.java.dev.jna:jna:5.11.0" + api "net.java.dev.jna:jna:5.13.0" api 'gradle.plugin.com.github.johnrengelman:shadow:7.1.2' api 'org.jdom:jdom2:2.0.6.1' api "org.jetbrains.kotlin:kotlin-stdlib-jdk8:${props.getProperty('kotlin')}" From b7e605d8011bd4de5fc040ee32333977ff154824 Mon Sep 17 00:00:00 2001 From: Andriy Redko Date: Mon, 30 Jan 2023 12:27:25 -0500 Subject: [PATCH 07/20] OpenJDK Update (January 2023 Patch releases) (#6074) Signed-off-by: Andriy Redko --- CHANGELOG.md | 1 + .../java/org/opensearch/gradle/test/DistroTestPlugin.java | 4 ++-- buildSrc/version.properties | 2 +- 3 files changed, 4 insertions(+), 3 deletions(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index 28d3fd9beb0f8..79e860900f870 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -38,6 +38,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Bumps `gson` from 2.10 to 2.10.1 - Bumps `json-schema-validator` from 1.0.73 to 1.0.76 - Bumps `jna` from 5.11.0 to 5.13.0 +- OpenJDK Update (January 2023 Patch releases) ([#6074](https://github.com/opensearch-project/OpenSearch/pull/6074)) ### Changed - [CCR] Add getHistoryOperationsFromTranslog method to fetch the history snapshot from translogs ([#3948](https://github.com/opensearch-project/OpenSearch/pull/3948)) diff --git a/buildSrc/src/main/java/org/opensearch/gradle/test/DistroTestPlugin.java b/buildSrc/src/main/java/org/opensearch/gradle/test/DistroTestPlugin.java index e7c907dfdf000..64e30700360b3 100644 --- a/buildSrc/src/main/java/org/opensearch/gradle/test/DistroTestPlugin.java +++ b/buildSrc/src/main/java/org/opensearch/gradle/test/DistroTestPlugin.java @@ -75,9 +75,9 @@ import java.util.stream.Stream; public class DistroTestPlugin implements Plugin { - private static final String SYSTEM_JDK_VERSION = "11.0.17+8"; + private static final String SYSTEM_JDK_VERSION = "11.0.18+10"; private static final String SYSTEM_JDK_VENDOR = "adoptium"; - private static final String GRADLE_JDK_VERSION = "17.0.5+8"; + private static final String GRADLE_JDK_VERSION = "17.0.6+10"; private static final String GRADLE_JDK_VENDOR = "adoptium"; // all distributions used by distro tests. this is temporary until tests are per distribution diff --git a/buildSrc/version.properties b/buildSrc/version.properties index 3b61c4a37327a..877bd5efba234 100644 --- a/buildSrc/version.properties +++ b/buildSrc/version.properties @@ -2,7 +2,7 @@ opensearch = 3.0.0 lucene = 9.5.0-snapshot-0878271 bundled_jdk_vendor = adoptium -bundled_jdk = 19.0.1+10 +bundled_jdk = 19.0.2+7 # optional dependencies From d6deb1f82f6f65c7c18ef6b09aa1dc3de09d34fe Mon Sep 17 00:00:00 2001 From: Suraj Singh Date: Mon, 30 Jan 2023 09:45:25 -0800 Subject: [PATCH 08/20] Mute testRelocateWhileContinuouslyIndexingAndWaitingForRefresh which times out on cluster health (#6079) Signed-off-by: Suraj Singh --- .../indices/replication/SegmentReplicationRelocationIT.java | 1 + 1 file changed, 1 insertion(+) diff --git a/server/src/internalClusterTest/java/org/opensearch/indices/replication/SegmentReplicationRelocationIT.java b/server/src/internalClusterTest/java/org/opensearch/indices/replication/SegmentReplicationRelocationIT.java index 5b0948dace75d..9cffe8c056f8a 100644 --- a/server/src/internalClusterTest/java/org/opensearch/indices/replication/SegmentReplicationRelocationIT.java +++ b/server/src/internalClusterTest/java/org/opensearch/indices/replication/SegmentReplicationRelocationIT.java @@ -215,6 +215,7 @@ public void testPrimaryRelocationWithSegRepFailure() throws Exception { * This test verifies primary recovery behavior with continuous ingestion * */ + @AwaitsFix(bugUrl = "https://github.com/opensearch-project/OpenSearch/issues/5669") public void testRelocateWhileContinuouslyIndexingAndWaitingForRefresh() throws Exception { final String primary = internalCluster().startNode(); createIndex(1); From 78d0d9614c423cfe720f31da6ca404d15181ce60 Mon Sep 17 00:00:00 2001 From: Andriy Redko Date: Mon, 30 Jan 2023 13:17:51 -0500 Subject: [PATCH 09/20] Bumps Mockito from 4.7.0 to 5.1.0, ByteBuddy from 1.12.18 to 1.12.22 (#6076) Signed-off-by: Andriy Redko --- CHANGELOG.md | 1 + buildSrc/version.properties | 5 ++--- client/rest/build.gradle | 1 + client/sniffer/build.gradle | 1 + 4 files changed, 5 insertions(+), 3 deletions(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index 79e860900f870..dd1bea96be8f2 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -39,6 +39,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Bumps `json-schema-validator` from 1.0.73 to 1.0.76 - Bumps `jna` from 5.11.0 to 5.13.0 - OpenJDK Update (January 2023 Patch releases) ([#6074](https://github.com/opensearch-project/OpenSearch/pull/6074)) +- Bumps `Mockito` from 4.7.0 to 5.1.0, `ByteBuddy` from 1.12.18 to 1.12.22 ([#6076](https://github.com/opensearch-project/OpenSearch/pull/6076)) ### Changed - [CCR] Add getHistoryOperationsFromTranslog method to fetch the history snapshot from translogs ([#3948](https://github.com/opensearch-project/OpenSearch/pull/3948)) diff --git a/buildSrc/version.properties b/buildSrc/version.properties index 877bd5efba234..a7b639ce1e933 100644 --- a/buildSrc/version.properties +++ b/buildSrc/version.properties @@ -48,10 +48,9 @@ bouncycastle=1.70 randomizedrunner = 2.7.1 junit = 4.13.2 hamcrest = 2.1 -# Update to 4.8.0 is using reflection without SecurityManager checks (fails with java.security.AccessControlException) -mockito = 4.7.0 +mockito = 5.1.0 objenesis = 3.2 -bytebuddy = 1.12.18 +bytebuddy = 1.12.22 # benchmark dependencies jmh = 1.35 diff --git a/client/rest/build.gradle b/client/rest/build.gradle index eacef14d17ce2..9ea7ad4ddb964 100644 --- a/client/rest/build.gradle +++ b/client/rest/build.gradle @@ -54,6 +54,7 @@ dependencies { testImplementation "org.mockito:mockito-core:${versions.mockito}" testImplementation "org.objenesis:objenesis:${versions.objenesis}" testImplementation "net.bytebuddy:byte-buddy:${versions.bytebuddy}" + testImplementation "net.bytebuddy:byte-buddy-agent:${versions.bytebuddy}" testImplementation "org.apache.logging.log4j:log4j-api:${versions.log4j}" testImplementation "org.apache.logging.log4j:log4j-core:${versions.log4j}" testImplementation "org.apache.logging.log4j:log4j-jul:${versions.log4j}" diff --git a/client/sniffer/build.gradle b/client/sniffer/build.gradle index eb3306cf2cea2..9823bc9afd347 100644 --- a/client/sniffer/build.gradle +++ b/client/sniffer/build.gradle @@ -50,6 +50,7 @@ dependencies { testImplementation "org.mockito:mockito-core:${versions.mockito}" testImplementation "org.objenesis:objenesis:${versions.objenesis}" testImplementation "net.bytebuddy:byte-buddy:${versions.bytebuddy}" + testImplementation "net.bytebuddy:byte-buddy-agent:${versions.bytebuddy}" } tasks.named('forbiddenApisMain').configure { From 4c9e92784b5e67c7f70c1c2c876f3da563345581 Mon Sep 17 00:00:00 2001 From: Marc Handalian Date: Mon, 30 Jan 2023 11:25:34 -0700 Subject: [PATCH 10/20] Fix broken template for issue autocuts on GH failure. (#6081) Signed-off-by: Marc Handalian --- .github/ISSUE_TEMPLATE/failed_check.md | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/.github/ISSUE_TEMPLATE/failed_check.md b/.github/ISSUE_TEMPLATE/failed_check.md index 86d90c2fd0efd..d6092a0404a06 100644 --- a/.github/ISSUE_TEMPLATE/failed_check.md +++ b/.github/ISSUE_TEMPLATE/failed_check.md @@ -1,6 +1,6 @@ --- -title: Gradle Check Failure. -labels: >test-failure bug +title: [AUTOCUT] Gradle Check Failure. +labels: >test-failure, bug --- A gradle check workflow has failed after merge. From 3e3b1551114b1d1fdb30b40bc7a38749ff1ecb3b Mon Sep 17 00:00:00 2001 From: Andriy Redko Date: Mon, 30 Jan 2023 14:03:49 -0500 Subject: [PATCH 11/20] Update Joda to 2.12.2 (#6083) Signed-off-by: Andriy Redko --- CHANGELOG.md | 1 + buildSrc/version.properties | 4 ++-- server/licenses/joda-time-2.10.13.jar.sha1 | 1 - server/licenses/joda-time-2.12.2.jar.sha1 | 1 + .../test/java/org/opensearch/common/time/DateUtilsTests.java | 2 +- 5 files changed, 5 insertions(+), 4 deletions(-) delete mode 100644 server/licenses/joda-time-2.10.13.jar.sha1 create mode 100644 server/licenses/joda-time-2.12.2.jar.sha1 diff --git a/CHANGELOG.md b/CHANGELOG.md index dd1bea96be8f2..c75db536dedac 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -40,6 +40,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Bumps `jna` from 5.11.0 to 5.13.0 - OpenJDK Update (January 2023 Patch releases) ([#6074](https://github.com/opensearch-project/OpenSearch/pull/6074)) - Bumps `Mockito` from 4.7.0 to 5.1.0, `ByteBuddy` from 1.12.18 to 1.12.22 ([#6076](https://github.com/opensearch-project/OpenSearch/pull/6076)) +- Bumps `joda` from 2.10.13 to 2.12.2 ([#6083](https://github.com/opensearch-project/OpenSearch/pull/6083)) ### Changed - [CCR] Add getHistoryOperationsFromTranslog method to fetch the history snapshot from translogs ([#3948](https://github.com/opensearch-project/OpenSearch/pull/3948)) diff --git a/buildSrc/version.properties b/buildSrc/version.properties index a7b639ce1e933..58fb0074d11f2 100644 --- a/buildSrc/version.properties +++ b/buildSrc/version.properties @@ -26,8 +26,8 @@ antlr4 = 4.11.1 jna = 5.5.0 netty = 4.1.86.Final -joda = 2.10.13 - +joda = 2.12.2 + # client dependencies httpclient5 = 5.1.4 httpcore5 = 5.1.5 diff --git a/server/licenses/joda-time-2.10.13.jar.sha1 b/server/licenses/joda-time-2.10.13.jar.sha1 deleted file mode 100644 index 4155902a4135d..0000000000000 --- a/server/licenses/joda-time-2.10.13.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -86f338c18cea2a89005556642e81707ff920dd38 \ No newline at end of file diff --git a/server/licenses/joda-time-2.12.2.jar.sha1 b/server/licenses/joda-time-2.12.2.jar.sha1 new file mode 100644 index 0000000000000..6e9b28eb35597 --- /dev/null +++ b/server/licenses/joda-time-2.12.2.jar.sha1 @@ -0,0 +1 @@ +78e18a7b4180e911dafba0a412adfa82c1e3d14b \ No newline at end of file diff --git a/server/src/test/java/org/opensearch/common/time/DateUtilsTests.java b/server/src/test/java/org/opensearch/common/time/DateUtilsTests.java index 95786a130106d..d9662d1de9e0c 100644 --- a/server/src/test/java/org/opensearch/common/time/DateUtilsTests.java +++ b/server/src/test/java/org/opensearch/common/time/DateUtilsTests.java @@ -58,7 +58,7 @@ import static org.hamcrest.Matchers.is; public class DateUtilsTests extends OpenSearchTestCase { - private static final Set IGNORE = new HashSet<>(Arrays.asList("Pacific/Enderbury", "Pacific/Kanton", "Pacific/Niue")); + private static final Set IGNORE = new HashSet<>(Arrays.asList("America/Ciudad_Juarez")); public void testTimezoneIds() { assertNull(DateUtils.dateTimeZoneToZoneId(null)); From 4dcadca26aa41ce1a2f02993b34acb6fdfd2e778 Mon Sep 17 00:00:00 2001 From: Marc Handalian Date: Mon, 30 Jan 2023 15:38:23 -0700 Subject: [PATCH 12/20] Fix autocut on GC failure caused by >test-failure label. (#6092) Signed-off-by: Marc Handalian --- .github/ISSUE_TEMPLATE/failed_check.md | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/.github/ISSUE_TEMPLATE/failed_check.md b/.github/ISSUE_TEMPLATE/failed_check.md index d6092a0404a06..ff216963cfffd 100644 --- a/.github/ISSUE_TEMPLATE/failed_check.md +++ b/.github/ISSUE_TEMPLATE/failed_check.md @@ -1,6 +1,6 @@ --- title: [AUTOCUT] Gradle Check Failure. -labels: >test-failure, bug +labels: ">test-failure, bug" --- A gradle check workflow has failed after merge. From 398598164def1327a3714ce895e5cf2345a4e7d6 Mon Sep 17 00:00:00 2001 From: "dependabot[bot]" <49699333+dependabot[bot]@users.noreply.github.com> Date: Mon, 30 Jan 2023 17:47:59 -0500 Subject: [PATCH 13/20] Bump joni from 2.1.44 to 2.1.45 in /libs/grok (#6070) * Bump joni from 2.1.44 to 2.1.45 in /libs/grok Bumps [joni](https://github.com/jruby/joni) from 2.1.44 to 2.1.45. - [Release notes](https://github.com/jruby/joni/releases) - [Commits](https://github.com/jruby/joni/compare/joni-2.1.44...joni-2.1.45) --- updated-dependencies: - dependency-name: org.jruby.joni:joni dependency-type: direct:production update-type: version-update:semver-patch ... Signed-off-by: dependabot[bot] * Updating SHAs Signed-off-by: dependabot[bot] * Update changelog Signed-off-by: dependabot[bot] --------- Signed-off-by: dependabot[bot] Co-authored-by: dependabot[bot] <49699333+dependabot[bot]@users.noreply.github.com> Co-authored-by: dependabot[bot] --- CHANGELOG.md | 1 + libs/grok/build.gradle | 2 +- libs/grok/licenses/joni-2.1.44.jar.sha1 | 1 - libs/grok/licenses/joni-2.1.45.jar.sha1 | 1 + 4 files changed, 3 insertions(+), 2 deletions(-) delete mode 100644 libs/grok/licenses/joni-2.1.44.jar.sha1 create mode 100644 libs/grok/licenses/joni-2.1.45.jar.sha1 diff --git a/CHANGELOG.md b/CHANGELOG.md index c75db536dedac..96de58446e85b 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -41,6 +41,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - OpenJDK Update (January 2023 Patch releases) ([#6074](https://github.com/opensearch-project/OpenSearch/pull/6074)) - Bumps `Mockito` from 4.7.0 to 5.1.0, `ByteBuddy` from 1.12.18 to 1.12.22 ([#6076](https://github.com/opensearch-project/OpenSearch/pull/6076)) - Bumps `joda` from 2.10.13 to 2.12.2 ([#6083](https://github.com/opensearch-project/OpenSearch/pull/6083)) +- Bumps `joni` from 2.1.44 to 2.1.45 ### Changed - [CCR] Add getHistoryOperationsFromTranslog method to fetch the history snapshot from translogs ([#3948](https://github.com/opensearch-project/OpenSearch/pull/3948)) diff --git a/libs/grok/build.gradle b/libs/grok/build.gradle index 43a55f84b9d55..057b83877d424 100644 --- a/libs/grok/build.gradle +++ b/libs/grok/build.gradle @@ -29,7 +29,7 @@ */ dependencies { - api 'org.jruby.joni:joni:2.1.44' + api 'org.jruby.joni:joni:2.1.45' // joni dependencies: api 'org.jruby.jcodings:jcodings:1.0.58' diff --git a/libs/grok/licenses/joni-2.1.44.jar.sha1 b/libs/grok/licenses/joni-2.1.44.jar.sha1 deleted file mode 100644 index bff9ca56f7e8c..0000000000000 --- a/libs/grok/licenses/joni-2.1.44.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -35746c2aee04ce459a2aa8dc2d626946c5dfb051 \ No newline at end of file diff --git a/libs/grok/licenses/joni-2.1.45.jar.sha1 b/libs/grok/licenses/joni-2.1.45.jar.sha1 new file mode 100644 index 0000000000000..8c9907b62217a --- /dev/null +++ b/libs/grok/licenses/joni-2.1.45.jar.sha1 @@ -0,0 +1 @@ +16f2d793ade587b04dd7050b47705d034faa144e \ No newline at end of file From 84b4145fa328459db69c0d1b965d5cb6981c7569 Mon Sep 17 00:00:00 2001 From: Joshua Palis Date: Mon, 30 Jan 2023 15:26:33 -0800 Subject: [PATCH 14/20] [Extensions] Changing ExtensionActionRequest streaminput constructor to be public (#6094) * Changing ExtensionActionRequest streaminput constructor to be public Signed-off-by: Joshua Palis * Updating changelog Signed-off-by: Joshua Palis * Moving changelog entry to unreleased 2.x Signed-off-by: Joshua Palis --------- Signed-off-by: Joshua Palis --- CHANGELOG.md | 1 + .../opensearch/extensions/action/ExtensionActionRequest.java | 2 +- 2 files changed, 2 insertions(+), 1 deletion(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index 96de58446e85b..fba2a25ccdeb3 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -85,6 +85,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Replace latches with CompletableFutures for extensions ([#5646](https://github.com/opensearch-project/OpenSearch/pull/5646)) - Add GeoTile and GeoHash Grid aggregations on GeoShapes. ([#5589](https://github.com/opensearch-project/OpenSearch/pull/5589)) - Add support to disallow search request with preference parameter with strict weighted shard routing([#5874](https://github.com/opensearch-project/OpenSearch/pull/5874)) +- Changing ExtensionActionRequest streaminput constructor to be public ([#6094](https://github.com/opensearch-project/OpenSearch/pull/6094)) ### Dependencies - Update nebula-publishing-plugin to 19.2.0 ([#5704](https://github.com/opensearch-project/OpenSearch/pull/5704)) diff --git a/server/src/main/java/org/opensearch/extensions/action/ExtensionActionRequest.java b/server/src/main/java/org/opensearch/extensions/action/ExtensionActionRequest.java index 801b40e847d21..dbcd682a38950 100644 --- a/server/src/main/java/org/opensearch/extensions/action/ExtensionActionRequest.java +++ b/server/src/main/java/org/opensearch/extensions/action/ExtensionActionRequest.java @@ -48,7 +48,7 @@ public ExtensionActionRequest(String action, byte[] requestBytes) { * @param in bytes stream input used to de-serialize the message. * @throws IOException when message de-serialization fails. */ - ExtensionActionRequest(StreamInput in) throws IOException { + public ExtensionActionRequest(StreamInput in) throws IOException { super(in); action = in.readString(); requestBytes = in.readByteArray(); From 1286a9438bb28fc0b8328f4380ca63d3f4b20a87 Mon Sep 17 00:00:00 2001 From: Marc Handalian Date: Mon, 30 Jan 2023 16:37:42 -0700 Subject: [PATCH 15/20] Remove CI related changelog entry. (#6098) Signed-off-by: Marc Handalian --- CHANGELOG.md | 3 +-- 1 file changed, 1 insertion(+), 2 deletions(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index fba2a25ccdeb3..1f5b1b50ac6ca 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -5,7 +5,6 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), ## [Unreleased 3.0] ### Added -- Hardened token permissions in GitHub workflows ([#4587](https://github.com/opensearch-project/OpenSearch/pull/4587)) - Support for HTTP/2 (server-side) ([#3847](https://github.com/opensearch-project/OpenSearch/pull/3847)) - Add getter for path field in NestedQueryBuilder ([#4636](https://github.com/opensearch-project/OpenSearch/pull/4636)) - Allow mmap to use new JDK-19 preview APIs in Apache Lucene 9.4+ ([#5151](https://github.com/opensearch-project/OpenSearch/pull/5151)) @@ -107,4 +106,4 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), ### Security [Unreleased 3.0]: https://github.com/opensearch-project/OpenSearch/compare/2.x...HEAD -[Unreleased 2.x]: https://github.com/opensearch-project/OpenSearch/compare/2.5...2.x \ No newline at end of file +[Unreleased 2.x]: https://github.com/opensearch-project/OpenSearch/compare/2.5...2.x From f8a3440c5279637eb6fe2a5989e1b27d7656cf7b Mon Sep 17 00:00:00 2001 From: Andrew Ross Date: Mon, 30 Jan 2023 16:32:30 -0800 Subject: [PATCH 16/20] Move dependency updates that have been backported (#6099) Signed-off-by: Andrew Ross --- CHANGELOG.md | 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) diff --git a/CHANGELOG.md b/CHANGELOG.md index 1f5b1b50ac6ca..a09e2ac686613 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -37,9 +37,6 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Bumps `gson` from 2.10 to 2.10.1 - Bumps `json-schema-validator` from 1.0.73 to 1.0.76 - Bumps `jna` from 5.11.0 to 5.13.0 -- OpenJDK Update (January 2023 Patch releases) ([#6074](https://github.com/opensearch-project/OpenSearch/pull/6074)) -- Bumps `Mockito` from 4.7.0 to 5.1.0, `ByteBuddy` from 1.12.18 to 1.12.22 ([#6076](https://github.com/opensearch-project/OpenSearch/pull/6076)) -- Bumps `joda` from 2.10.13 to 2.12.2 ([#6083](https://github.com/opensearch-project/OpenSearch/pull/6083)) - Bumps `joni` from 2.1.44 to 2.1.45 ### Changed @@ -89,6 +86,9 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), ### Dependencies - Update nebula-publishing-plugin to 19.2.0 ([#5704](https://github.com/opensearch-project/OpenSearch/pull/5704)) - Bumps `reactor-netty` from 1.1.1 to 1.1.2 ([#5878](https://github.com/opensearch-project/OpenSearch/pull/5878)) +- OpenJDK Update (January 2023 Patch releases) ([#6074](https://github.com/opensearch-project/OpenSearch/pull/6074)) +- Bumps `Mockito` from 4.7.0 to 5.1.0, `ByteBuddy` from 1.12.18 to 1.12.22 ([#6076](https://github.com/opensearch-project/OpenSearch/pull/6076)) +- Bumps `joda` from 2.10.13 to 2.12.2 ([#6083](https://github.com/opensearch-project/OpenSearch/pull/6083)) ### Changed - Use ReplicationFailedException instead of OpensearchException in ReplicationTarget ([#4725](https://github.com/opensearch-project/OpenSearch/pull/4725)) From 7aea7d2506644cc6316c33f16c2acc1046fd07d2 Mon Sep 17 00:00:00 2001 From: Marc Handalian Date: Mon, 30 Jan 2023 18:29:17 -0700 Subject: [PATCH 17/20] Fix AUTOCUT title formatting in failed_check.md (#6100) Signed-off-by: Marc Handalian --- .github/ISSUE_TEMPLATE/failed_check.md | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/.github/ISSUE_TEMPLATE/failed_check.md b/.github/ISSUE_TEMPLATE/failed_check.md index ff216963cfffd..24a323b7dd0f7 100644 --- a/.github/ISSUE_TEMPLATE/failed_check.md +++ b/.github/ISSUE_TEMPLATE/failed_check.md @@ -1,6 +1,6 @@ --- -title: [AUTOCUT] Gradle Check Failure. -labels: ">test-failure, bug" +title: '[AUTOCUT] Gradle Check Failure.' +labels: '>test-failure, bug' --- A gradle check workflow has failed after merge. From c557f2717ad45627cacd88e8243893dd84a56623 Mon Sep 17 00:00:00 2001 From: Nick Knize Date: Tue, 31 Jan 2023 09:45:39 -0600 Subject: [PATCH 18/20] [Refactor] core.common to new opensearch-common library (#5976) Refactors all of o.opensearch.common classes in the core library to a new opensearch-commons library (inspired by apache commons). The intent is to refactor any o.opensearch.common.* classes in :server module to this new library. This will be done with care such that the dependencies are carefully managed to avoid any cyclic or unnecessarily complicated dependencies. Signed-off-by: Nicholas Walter Knize --- CHANGELOG.md | 1 + buildSrc/build.gradle | 1 + .../precommit/JarHellPrecommitPlugin.java | 4 +- gradle/missing-javadoc.gradle | 7 +++- libs/build.gradle | 5 ++- libs/cli/build.gradle | 1 + libs/common/build.gradle | 35 ++++++++++++++++++ .../org/opensearch/bootstrap/JarHell.java | 2 + .../opensearch/bootstrap/JdkJarHellCheck.java | 5 +++ .../opensearch/bootstrap/package-info.java | 10 +++++ .../java/org/opensearch/common/Booleans.java | 5 +++ .../org/opensearch/common/CharArrays.java | 2 + .../opensearch/common/CheckedConsumer.java | 2 + .../opensearch/common/CheckedFunction.java | 2 + .../opensearch/common/CheckedRunnable.java | 2 + .../main/java/org/opensearch/common/Glob.java | 2 + .../opensearch/common/MemoizedSupplier.java | 5 +++ .../java/org/opensearch/common/Nullable.java | 2 +- .../java/org/opensearch/common/SetOnce.java | 0 .../opensearch/common/SuppressForbidden.java | 2 + .../org/opensearch/common/collect/List.java | 7 ++++ .../org/opensearch/common/collect/Map.java | 7 ++++ .../org/opensearch/common/collect/Set.java | 7 ++++ .../org/opensearch/common/collect/Tuple.java | 7 ++++ .../common/collect/package-info.java | 10 +++++ .../common/concurrent/CompletableContext.java | 2 + .../common/concurrent/package-info.java | 10 +++++ .../org/opensearch/common/io/PathUtils.java | 2 + .../opensearch/common/io/package-info.java | 10 +++++ .../org/opensearch/common/package-info.java | 10 +++++ .../org/opensearch/common/unit/TimeValue.java | 5 +++ .../opensearch/common/unit/package-info.java | 10 +++++ .../org/opensearch/common/util/FastMath.java | 2 + .../common/util/OpenSearchSloppyMath.java | 2 + .../util/concurrent/AbstractRefCounted.java | 2 + .../common/util/concurrent/RefCounted.java | 2 + .../common/util/concurrent/package-info.java | 10 +++++ .../opensearch/common/util/package-info.java | 10 +++++ .../java/org/opensearch/package-info.java | 10 +++++ .../opensearch/bootstrap/JarHellTests.java | 0 .../opensearch/common/CharArraysTests.java | 0 .../org/opensearch/common/SetOnceTests.java | 0 .../opensearch/common/collect/ListTests.java | 0 .../opensearch/common/collect/MapTests.java | 0 .../opensearch/common/collect/SetTests.java | 0 .../opensearch/common/collect/TupleTests.java | 0 .../common/unit/TimeValueTests.java | 0 .../util/OpenSearchSloppyMathTests.java | 0 .../util/concurrent/RefCountedTests.java | 0 .../bootstrap/duplicate-classes.jar | Bin .../bootstrap/duplicate-xmlbeans-classes.jar | Bin libs/core/build.gradle | 3 +- .../opensearch/core/internal/io/IOUtils.java | 2 + .../opensearch/core/internal/io/Streams.java | 2 + .../core/internal/io/package-info.java | 10 +++++ .../core/internal/net/NetUtils.java | 2 + .../core/internal/net/package-info.java | 10 +++++ libs/nio/build.gradle | 3 +- libs/ssl-config/build.gradle | 3 +- libs/x-content/build.gradle | 1 + .../common/logging/EvilLoggerTests.java | 3 -- qa/os/build.gradle | 1 + server/build.gradle | 1 + .../monitor/process/ProcessStats.java | 1 - .../node/AdaptiveSelectionStats.java | 1 - .../org/opensearch/search/SearchService.java | 4 -- 66 files changed, 247 insertions(+), 20 deletions(-) create mode 100644 libs/common/build.gradle rename libs/{core => common}/src/main/java/org/opensearch/bootstrap/JarHell.java (99%) rename libs/{core => common}/src/main/java/org/opensearch/bootstrap/JdkJarHellCheck.java (98%) create mode 100644 libs/common/src/main/java/org/opensearch/bootstrap/package-info.java rename libs/{core => common}/src/main/java/org/opensearch/common/Booleans.java (99%) rename libs/{core => common}/src/main/java/org/opensearch/common/CharArrays.java (99%) rename libs/{core => common}/src/main/java/org/opensearch/common/CheckedConsumer.java (98%) rename libs/{core => common}/src/main/java/org/opensearch/common/CheckedFunction.java (98%) rename libs/{core => common}/src/main/java/org/opensearch/common/CheckedRunnable.java (98%) rename libs/{core => common}/src/main/java/org/opensearch/common/Glob.java (99%) rename libs/{core => common}/src/main/java/org/opensearch/common/MemoizedSupplier.java (94%) rename libs/{core => common}/src/main/java/org/opensearch/common/Nullable.java (98%) rename libs/{core => common}/src/main/java/org/opensearch/common/SetOnce.java (100%) rename libs/{core => common}/src/main/java/org/opensearch/common/SuppressForbidden.java (98%) rename libs/{core => common}/src/main/java/org/opensearch/common/collect/List.java (96%) rename libs/{core => common}/src/main/java/org/opensearch/common/collect/Map.java (97%) rename libs/{core => common}/src/main/java/org/opensearch/common/collect/Set.java (96%) rename libs/{core => common}/src/main/java/org/opensearch/common/collect/Tuple.java (95%) create mode 100644 libs/common/src/main/java/org/opensearch/common/collect/package-info.java rename libs/{core => common}/src/main/java/org/opensearch/common/concurrent/CompletableContext.java (99%) create mode 100644 libs/common/src/main/java/org/opensearch/common/concurrent/package-info.java rename libs/{core => common}/src/main/java/org/opensearch/common/io/PathUtils.java (99%) create mode 100644 libs/common/src/main/java/org/opensearch/common/io/package-info.java create mode 100644 libs/common/src/main/java/org/opensearch/common/package-info.java rename libs/{core => common}/src/main/java/org/opensearch/common/unit/TimeValue.java (99%) create mode 100644 libs/common/src/main/java/org/opensearch/common/unit/package-info.java rename libs/{core => common}/src/main/java/org/opensearch/common/util/FastMath.java (99%) rename libs/{core => common}/src/main/java/org/opensearch/common/util/OpenSearchSloppyMath.java (98%) rename libs/{core => common}/src/main/java/org/opensearch/common/util/concurrent/AbstractRefCounted.java (99%) rename libs/{core => common}/src/main/java/org/opensearch/common/util/concurrent/RefCounted.java (99%) create mode 100644 libs/common/src/main/java/org/opensearch/common/util/concurrent/package-info.java create mode 100644 libs/common/src/main/java/org/opensearch/common/util/package-info.java create mode 100644 libs/common/src/main/java/org/opensearch/package-info.java rename libs/{core => common}/src/test/java/org/opensearch/bootstrap/JarHellTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/CharArraysTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/SetOnceTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/collect/ListTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/collect/MapTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/collect/SetTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/collect/TupleTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/unit/TimeValueTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/util/OpenSearchSloppyMathTests.java (100%) rename libs/{core => common}/src/test/java/org/opensearch/common/util/concurrent/RefCountedTests.java (100%) rename libs/{core => common}/src/test/resources/org/opensearch/bootstrap/duplicate-classes.jar (100%) rename libs/{core => common}/src/test/resources/org/opensearch/bootstrap/duplicate-xmlbeans-classes.jar (100%) create mode 100644 libs/core/src/main/java/org/opensearch/core/internal/io/package-info.java create mode 100644 libs/core/src/main/java/org/opensearch/core/internal/net/package-info.java diff --git a/CHANGELOG.md b/CHANGELOG.md index a09e2ac686613..2d3c8fa81cc75 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -94,6 +94,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - Use ReplicationFailedException instead of OpensearchException in ReplicationTarget ([#4725](https://github.com/opensearch-project/OpenSearch/pull/4725)) - [Refactor] Use local opensearch.common.SetOnce instead of lucene's utility class ([#5947](https://github.com/opensearch-project/OpenSearch/pull/5947)) - Cluster health call to throw decommissioned exception for local decommissioned node([#6008](https://github.com/opensearch-project/OpenSearch/pull/6008)) +- [Refactor] core.common to new opensearch-common library ([#5976](https://github.com/opensearch-project/OpenSearch/pull/5976)) ### Deprecated diff --git a/buildSrc/build.gradle b/buildSrc/build.gradle index 070c4493ee687..3dd5682f499cc 100644 --- a/buildSrc/build.gradle +++ b/buildSrc/build.gradle @@ -191,6 +191,7 @@ if (project != rootProject) { distribution project(':distribution:archives:linux-tar') distribution project(':distribution:archives:windows-zip') + integTestRuntimeOnly(project(":libs:opensearch-common")) integTestRuntimeOnly(project(":libs:opensearch-core")) } diff --git a/buildSrc/src/main/java/org/opensearch/gradle/precommit/JarHellPrecommitPlugin.java b/buildSrc/src/main/java/org/opensearch/gradle/precommit/JarHellPrecommitPlugin.java index 87bf03cec3587..429028c9bf841 100644 --- a/buildSrc/src/main/java/org/opensearch/gradle/precommit/JarHellPrecommitPlugin.java +++ b/buildSrc/src/main/java/org/opensearch/gradle/precommit/JarHellPrecommitPlugin.java @@ -44,11 +44,11 @@ public class JarHellPrecommitPlugin extends PrecommitPlugin { @Override public TaskProvider createTask(Project project) { Configuration jarHellConfig = project.getConfigurations().create("jarHell"); - if (BuildParams.isInternal() && project.getPath().equals(":libs:opensearch-core") == false) { + if (BuildParams.isInternal() && project.getPath().equals(":libs:opensearch-common") == false) { // External plugins will depend on this already via transitive dependencies. // Internal projects are not all plugins, so make sure the check is available // we are not doing this for this project itself to avoid jar hell with itself - project.getDependencies().add("jarHell", project.project(":libs:opensearch-core")); + project.getDependencies().add("jarHell", project.project(":libs:opensearch-common")); } TaskProvider jarHell = project.getTasks().register("jarHell", JarHellTask.class); diff --git a/gradle/missing-javadoc.gradle b/gradle/missing-javadoc.gradle index a1fde7637796c..98619f4722afa 100644 --- a/gradle/missing-javadoc.gradle +++ b/gradle/missing-javadoc.gradle @@ -96,7 +96,6 @@ configure([ project(":client:rest-high-level"), project(":client:test"), project(":libs:opensearch-cli"), - project(":libs:opensearch-core"), project(":libs:opensearch-dissect"), project(":libs:opensearch-geo"), project(":libs:opensearch-grok"), @@ -162,7 +161,11 @@ configure([ } } -configure(project(":server")) { +configure([ + project(":libs:opensearch-common"), + project(":libs:opensearch-core"), + project(":server") +]) { project.tasks.withType(MissingJavadocTask) { // TODO: bump to variable missing level after increasing javadoc coverage javadocMissingLevel = "class" diff --git a/libs/build.gradle b/libs/build.gradle index 92cb2bdb2efc7..39d2737966b6d 100644 --- a/libs/build.gradle +++ b/libs/build.gradle @@ -44,11 +44,12 @@ subprojects { dependencies.matching { it instanceof ProjectDependency }.all { ProjectDependency dep -> Project depProject = dep.dependencyProject if (depProject != null - && false == depProject.path.equals(':libs:opensearch-core') + && (false == depProject.path.equals(':libs:opensearch-core') && + false == depProject.path.equals(':libs:opensearch-common')) && depProject.path.startsWith(':libs')) { throw new InvalidUserDataException("projects in :libs " + "may not depend on other projects libs except " - + ":libs:opensearch-core but " + + ":libs:opensearch-core or :libs:opensearch-common but " + "${project.path} depends on ${depProject.path}") } } diff --git a/libs/cli/build.gradle b/libs/cli/build.gradle index bbb7bf68e2ced..2ce9cc86cf69a 100644 --- a/libs/cli/build.gradle +++ b/libs/cli/build.gradle @@ -33,6 +33,7 @@ apply plugin: 'opensearch.publish' dependencies { api 'net.sf.jopt-simple:jopt-simple:5.0.4' + api project(':libs:opensearch-common') api project(':libs:opensearch-core') } diff --git a/libs/common/build.gradle b/libs/common/build.gradle new file mode 100644 index 0000000000000..a2356a806174c --- /dev/null +++ b/libs/common/build.gradle @@ -0,0 +1,35 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + * + * Modifications Copyright OpenSearch Contributors. See + * GitHub history for details. + */ + +import org.opensearch.gradle.info.BuildParams + +apply plugin: 'opensearch.publish' + +archivesBaseName = 'opensearch-common' + +dependencies { + // This dependency is used only by :libs:core for null-checking interop with other tools + compileOnly "com.google.code.findbugs:jsr305:3.0.2" + + testImplementation "com.carrotsearch.randomizedtesting:randomizedtesting-runner:${versions.randomizedrunner}" + testImplementation "junit:junit:${versions.junit}" + testImplementation "org.hamcrest:hamcrest:${versions.hamcrest}" + + testImplementation(project(":test:framework")) { + exclude group: 'org.opensearch', module: 'opensearch-common' + } +} + +tasks.named('forbiddenApisMain').configure { + // :libs:opensearch-common does not depend on server + // TODO: Need to decide how we want to handle for forbidden signatures with the changes to server + replaceSignatureFiles 'jdk-signatures' +} diff --git a/libs/core/src/main/java/org/opensearch/bootstrap/JarHell.java b/libs/common/src/main/java/org/opensearch/bootstrap/JarHell.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/bootstrap/JarHell.java rename to libs/common/src/main/java/org/opensearch/bootstrap/JarHell.java index d945697b21c0b..c4ba778e7db86 100644 --- a/libs/core/src/main/java/org/opensearch/bootstrap/JarHell.java +++ b/libs/common/src/main/java/org/opensearch/bootstrap/JarHell.java @@ -71,6 +71,8 @@ *
  • Checks any {@code X-Compile-OpenSearch-Version} value in * the jar manifest is compatible with the current ES
  • * + * + * @opensearch.internal */ public class JarHell { diff --git a/libs/core/src/main/java/org/opensearch/bootstrap/JdkJarHellCheck.java b/libs/common/src/main/java/org/opensearch/bootstrap/JdkJarHellCheck.java similarity index 98% rename from libs/core/src/main/java/org/opensearch/bootstrap/JdkJarHellCheck.java rename to libs/common/src/main/java/org/opensearch/bootstrap/JdkJarHellCheck.java index 8cb81639c504f..54c87bdceddc2 100644 --- a/libs/core/src/main/java/org/opensearch/bootstrap/JdkJarHellCheck.java +++ b/libs/common/src/main/java/org/opensearch/bootstrap/JdkJarHellCheck.java @@ -44,6 +44,11 @@ import java.util.HashSet; import java.util.Set; +/** + * Checks for jdk jar hell + * + * @opensearch.internal + */ public class JdkJarHellCheck { private Set detected = new HashSet<>(); diff --git a/libs/common/src/main/java/org/opensearch/bootstrap/package-info.java b/libs/common/src/main/java/org/opensearch/bootstrap/package-info.java new file mode 100644 index 0000000000000..f522b1bb91444 --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/bootstrap/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Contains JarHell Classes */ +package org.opensearch.bootstrap; diff --git a/libs/core/src/main/java/org/opensearch/common/Booleans.java b/libs/common/src/main/java/org/opensearch/common/Booleans.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/Booleans.java rename to libs/common/src/main/java/org/opensearch/common/Booleans.java index 70536a8f4cc46..4fbaa78921cde 100644 --- a/libs/core/src/main/java/org/opensearch/common/Booleans.java +++ b/libs/common/src/main/java/org/opensearch/common/Booleans.java @@ -32,6 +32,11 @@ package org.opensearch.common; +/** + * Boolean class utilities + * + * @opensearch.api + */ public final class Booleans { private Booleans() { throw new AssertionError("No instances intended"); diff --git a/libs/core/src/main/java/org/opensearch/common/CharArrays.java b/libs/common/src/main/java/org/opensearch/common/CharArrays.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/CharArrays.java rename to libs/common/src/main/java/org/opensearch/common/CharArrays.java index b579581b7044d..5489c067fc887 100644 --- a/libs/core/src/main/java/org/opensearch/common/CharArrays.java +++ b/libs/common/src/main/java/org/opensearch/common/CharArrays.java @@ -40,6 +40,8 @@ /** * Helper class similar to Arrays to handle conversions for Char arrays + * + * @opensearch.api */ public final class CharArrays { diff --git a/libs/core/src/main/java/org/opensearch/common/CheckedConsumer.java b/libs/common/src/main/java/org/opensearch/common/CheckedConsumer.java similarity index 98% rename from libs/core/src/main/java/org/opensearch/common/CheckedConsumer.java rename to libs/common/src/main/java/org/opensearch/common/CheckedConsumer.java index d52c69ee517f1..dede06d0e207d 100644 --- a/libs/core/src/main/java/org/opensearch/common/CheckedConsumer.java +++ b/libs/common/src/main/java/org/opensearch/common/CheckedConsumer.java @@ -36,6 +36,8 @@ /** * A {@link Consumer}-like interface which allows throwing checked exceptions. + * + * @opensearch.api */ @FunctionalInterface public interface CheckedConsumer { diff --git a/libs/core/src/main/java/org/opensearch/common/CheckedFunction.java b/libs/common/src/main/java/org/opensearch/common/CheckedFunction.java similarity index 98% rename from libs/core/src/main/java/org/opensearch/common/CheckedFunction.java rename to libs/common/src/main/java/org/opensearch/common/CheckedFunction.java index 651f68b17a8d8..9c17ad4b4ee3f 100644 --- a/libs/core/src/main/java/org/opensearch/common/CheckedFunction.java +++ b/libs/common/src/main/java/org/opensearch/common/CheckedFunction.java @@ -36,6 +36,8 @@ /** * A {@link Function}-like interface which allows throwing checked exceptions. + * + * @opensearch.api */ @FunctionalInterface public interface CheckedFunction { diff --git a/libs/core/src/main/java/org/opensearch/common/CheckedRunnable.java b/libs/common/src/main/java/org/opensearch/common/CheckedRunnable.java similarity index 98% rename from libs/core/src/main/java/org/opensearch/common/CheckedRunnable.java rename to libs/common/src/main/java/org/opensearch/common/CheckedRunnable.java index ac1b2b82ade8e..cb773ab789180 100644 --- a/libs/core/src/main/java/org/opensearch/common/CheckedRunnable.java +++ b/libs/common/src/main/java/org/opensearch/common/CheckedRunnable.java @@ -34,6 +34,8 @@ /** * A {@link Runnable}-like interface which allows throwing checked exceptions. + * + * @opensearch.api */ @FunctionalInterface public interface CheckedRunnable { diff --git a/libs/core/src/main/java/org/opensearch/common/Glob.java b/libs/common/src/main/java/org/opensearch/common/Glob.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/Glob.java rename to libs/common/src/main/java/org/opensearch/common/Glob.java index 650a38b83fbfc..daf045dd49e3a 100644 --- a/libs/core/src/main/java/org/opensearch/common/Glob.java +++ b/libs/common/src/main/java/org/opensearch/common/Glob.java @@ -34,6 +34,8 @@ /** * Utility class for glob-like matching + * + * @opensearch.api */ public class Glob { diff --git a/libs/core/src/main/java/org/opensearch/common/MemoizedSupplier.java b/libs/common/src/main/java/org/opensearch/common/MemoizedSupplier.java similarity index 94% rename from libs/core/src/main/java/org/opensearch/common/MemoizedSupplier.java rename to libs/common/src/main/java/org/opensearch/common/MemoizedSupplier.java index 590dae2323abd..0bfdcf6b7a144 100644 --- a/libs/core/src/main/java/org/opensearch/common/MemoizedSupplier.java +++ b/libs/common/src/main/java/org/opensearch/common/MemoizedSupplier.java @@ -34,6 +34,11 @@ import java.util.function.Supplier; +/** + * A base supplier using memoization optimization technique + * + * @opensearch.api + */ public class MemoizedSupplier implements Supplier { private Supplier supplier; private T value; diff --git a/libs/core/src/main/java/org/opensearch/common/Nullable.java b/libs/common/src/main/java/org/opensearch/common/Nullable.java similarity index 98% rename from libs/core/src/main/java/org/opensearch/common/Nullable.java rename to libs/common/src/main/java/org/opensearch/common/Nullable.java index 8660f2dbb5995..804b339449147 100644 --- a/libs/core/src/main/java/org/opensearch/common/Nullable.java +++ b/libs/common/src/main/java/org/opensearch/common/Nullable.java @@ -45,7 +45,7 @@ * {@code null} is an acceptable value for that parameter. It should not be * used for parameters of primitive types. * - * + * @opensearch.api */ @Documented @TypeQualifierNickname diff --git a/libs/core/src/main/java/org/opensearch/common/SetOnce.java b/libs/common/src/main/java/org/opensearch/common/SetOnce.java similarity index 100% rename from libs/core/src/main/java/org/opensearch/common/SetOnce.java rename to libs/common/src/main/java/org/opensearch/common/SetOnce.java diff --git a/libs/core/src/main/java/org/opensearch/common/SuppressForbidden.java b/libs/common/src/main/java/org/opensearch/common/SuppressForbidden.java similarity index 98% rename from libs/core/src/main/java/org/opensearch/common/SuppressForbidden.java rename to libs/common/src/main/java/org/opensearch/common/SuppressForbidden.java index ed4045dd1ef3c..1f1b28bcf6759 100644 --- a/libs/core/src/main/java/org/opensearch/common/SuppressForbidden.java +++ b/libs/common/src/main/java/org/opensearch/common/SuppressForbidden.java @@ -38,6 +38,8 @@ /** * Annotation to suppress forbidden-apis errors inside a whole class, a method, or a field. + * + * @opensearch.api */ @Retention(RetentionPolicy.CLASS) @Target({ ElementType.CONSTRUCTOR, ElementType.FIELD, ElementType.METHOD, ElementType.TYPE }) diff --git a/libs/core/src/main/java/org/opensearch/common/collect/List.java b/libs/common/src/main/java/org/opensearch/common/collect/List.java similarity index 96% rename from libs/core/src/main/java/org/opensearch/common/collect/List.java rename to libs/common/src/main/java/org/opensearch/common/collect/List.java index 56216d6bbafe2..c28a69a515d35 100644 --- a/libs/core/src/main/java/org/opensearch/common/collect/List.java +++ b/libs/common/src/main/java/org/opensearch/common/collect/List.java @@ -34,6 +34,13 @@ import java.util.Collection; +/** + * Java 9 List + * + * todo: deprecate and remove w/ min jdk upgrade to 11? + * + * @opensearch.internal + */ public class List { /** diff --git a/libs/core/src/main/java/org/opensearch/common/collect/Map.java b/libs/common/src/main/java/org/opensearch/common/collect/Map.java similarity index 97% rename from libs/core/src/main/java/org/opensearch/common/collect/Map.java rename to libs/common/src/main/java/org/opensearch/common/collect/Map.java index 21de546869390..a0b8c03d3d3e4 100644 --- a/libs/core/src/main/java/org/opensearch/common/collect/Map.java +++ b/libs/common/src/main/java/org/opensearch/common/collect/Map.java @@ -32,6 +32,13 @@ package org.opensearch.common.collect; +/** + * Java 9 Map + * + * todo: deprecate and remove w/ min jdk upgrade to 11? + * + * @opensearch.internal + */ public class Map { /** diff --git a/libs/core/src/main/java/org/opensearch/common/collect/Set.java b/libs/common/src/main/java/org/opensearch/common/collect/Set.java similarity index 96% rename from libs/core/src/main/java/org/opensearch/common/collect/Set.java rename to libs/common/src/main/java/org/opensearch/common/collect/Set.java index 0350023e4e894..11d59cead6009 100644 --- a/libs/core/src/main/java/org/opensearch/common/collect/Set.java +++ b/libs/common/src/main/java/org/opensearch/common/collect/Set.java @@ -34,6 +34,13 @@ import java.util.Collection; +/** + * Java 9 Set + * + * todo: deprecate and remove w/ min jdk upgrade to 11? + * + * @opensearch.internal + */ public class Set { /** diff --git a/libs/core/src/main/java/org/opensearch/common/collect/Tuple.java b/libs/common/src/main/java/org/opensearch/common/collect/Tuple.java similarity index 95% rename from libs/core/src/main/java/org/opensearch/common/collect/Tuple.java rename to libs/common/src/main/java/org/opensearch/common/collect/Tuple.java index cc82123056423..36bc5504061f5 100644 --- a/libs/core/src/main/java/org/opensearch/common/collect/Tuple.java +++ b/libs/common/src/main/java/org/opensearch/common/collect/Tuple.java @@ -32,6 +32,13 @@ package org.opensearch.common.collect; +/** + * Java 9 Tuple + * + * todo: deprecate and remove w/ min jdk upgrade to 11? + * + * @opensearch.internal + */ public class Tuple { public static Tuple tuple(V1 v1, V2 v2) { diff --git a/libs/common/src/main/java/org/opensearch/common/collect/package-info.java b/libs/common/src/main/java/org/opensearch/common/collect/package-info.java new file mode 100644 index 0000000000000..d08cfad1d178b --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/common/collect/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Common collections classes used across opensearch. */ +package org.opensearch.common.collect; diff --git a/libs/core/src/main/java/org/opensearch/common/concurrent/CompletableContext.java b/libs/common/src/main/java/org/opensearch/common/concurrent/CompletableContext.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/concurrent/CompletableContext.java rename to libs/common/src/main/java/org/opensearch/common/concurrent/CompletableContext.java index 8565461cc74bd..f7910af51b599 100644 --- a/libs/core/src/main/java/org/opensearch/common/concurrent/CompletableContext.java +++ b/libs/common/src/main/java/org/opensearch/common/concurrent/CompletableContext.java @@ -41,6 +41,8 @@ * an exceptional result. This allows attaching listeners that only handle {@link Exception}. * * @param the result type + * + * @opensearch.api */ public class CompletableContext { diff --git a/libs/common/src/main/java/org/opensearch/common/concurrent/package-info.java b/libs/common/src/main/java/org/opensearch/common/concurrent/package-info.java new file mode 100644 index 0000000000000..d412fdfa4f5dc --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/common/concurrent/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Common concurrency utilities used across opensearch. */ +package org.opensearch.common.concurrent; diff --git a/libs/core/src/main/java/org/opensearch/common/io/PathUtils.java b/libs/common/src/main/java/org/opensearch/common/io/PathUtils.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/io/PathUtils.java rename to libs/common/src/main/java/org/opensearch/common/io/PathUtils.java index 85b623ef4d06d..b3526859933ec 100644 --- a/libs/core/src/main/java/org/opensearch/common/io/PathUtils.java +++ b/libs/common/src/main/java/org/opensearch/common/io/PathUtils.java @@ -46,6 +46,8 @@ *

    * This class allows the default filesystem to * be changed during tests. + * + * @opensearch.internal */ @SuppressForbidden(reason = "accesses the default filesystem by design") // TODO: can we move this to the .env package and make it package-private? diff --git a/libs/common/src/main/java/org/opensearch/common/io/package-info.java b/libs/common/src/main/java/org/opensearch/common/io/package-info.java new file mode 100644 index 0000000000000..e79aeb830761b --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/common/io/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Common i/o utilities used across opensearch. */ +package org.opensearch.common.io; diff --git a/libs/common/src/main/java/org/opensearch/common/package-info.java b/libs/common/src/main/java/org/opensearch/common/package-info.java new file mode 100644 index 0000000000000..f91bac5192f63 --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/common/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Common Library of Utilties and Data Structures used across OpenSearch. */ +package org.opensearch.common; diff --git a/libs/core/src/main/java/org/opensearch/common/unit/TimeValue.java b/libs/common/src/main/java/org/opensearch/common/unit/TimeValue.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/unit/TimeValue.java rename to libs/common/src/main/java/org/opensearch/common/unit/TimeValue.java index ee150bf4ff85a..670275397893c 100644 --- a/libs/core/src/main/java/org/opensearch/common/unit/TimeValue.java +++ b/libs/common/src/main/java/org/opensearch/common/unit/TimeValue.java @@ -36,6 +36,11 @@ import java.util.Objects; import java.util.concurrent.TimeUnit; +/** + * Time value unit of measurement + * + * @opensearch.api + */ public class TimeValue implements Comparable { /** How many nano-seconds in one milli-second */ diff --git a/libs/common/src/main/java/org/opensearch/common/unit/package-info.java b/libs/common/src/main/java/org/opensearch/common/unit/package-info.java new file mode 100644 index 0000000000000..f2387c4349cc5 --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/common/unit/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Common units of measurement used across opensearch. */ +package org.opensearch.common.unit; diff --git a/libs/core/src/main/java/org/opensearch/common/util/FastMath.java b/libs/common/src/main/java/org/opensearch/common/util/FastMath.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/util/FastMath.java rename to libs/common/src/main/java/org/opensearch/common/util/FastMath.java index f3b04fda2cadb..ac95a39698787 100644 --- a/libs/core/src/main/java/org/opensearch/common/util/FastMath.java +++ b/libs/common/src/main/java/org/opensearch/common/util/FastMath.java @@ -47,6 +47,8 @@ /** * Additions or modifications to this class should only come from the original org.math.plot.utils.FastMath source + * + * @opensearch.api */ final class FastMath { diff --git a/libs/core/src/main/java/org/opensearch/common/util/OpenSearchSloppyMath.java b/libs/common/src/main/java/org/opensearch/common/util/OpenSearchSloppyMath.java similarity index 98% rename from libs/core/src/main/java/org/opensearch/common/util/OpenSearchSloppyMath.java rename to libs/common/src/main/java/org/opensearch/common/util/OpenSearchSloppyMath.java index 4c24e87fe74b4..154b9c3a3ffc8 100644 --- a/libs/core/src/main/java/org/opensearch/common/util/OpenSearchSloppyMath.java +++ b/libs/common/src/main/java/org/opensearch/common/util/OpenSearchSloppyMath.java @@ -34,6 +34,8 @@ /** * Similar to Lucene's SloppyMath, but for additional math functions. + * + * @opensearch.api */ public class OpenSearchSloppyMath { diff --git a/libs/core/src/main/java/org/opensearch/common/util/concurrent/AbstractRefCounted.java b/libs/common/src/main/java/org/opensearch/common/util/concurrent/AbstractRefCounted.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/util/concurrent/AbstractRefCounted.java rename to libs/common/src/main/java/org/opensearch/common/util/concurrent/AbstractRefCounted.java index e55c87c34c795..2e23e87bea534 100644 --- a/libs/core/src/main/java/org/opensearch/common/util/concurrent/AbstractRefCounted.java +++ b/libs/common/src/main/java/org/opensearch/common/util/concurrent/AbstractRefCounted.java @@ -38,6 +38,8 @@ * A basic RefCounted implementation that is initialized with a * ref count of 1 and calls {@link #closeInternal()} once it reaches * a 0 ref count + * + * @opensearch.api */ public abstract class AbstractRefCounted implements RefCounted { private final AtomicInteger refCount = new AtomicInteger(1); diff --git a/libs/core/src/main/java/org/opensearch/common/util/concurrent/RefCounted.java b/libs/common/src/main/java/org/opensearch/common/util/concurrent/RefCounted.java similarity index 99% rename from libs/core/src/main/java/org/opensearch/common/util/concurrent/RefCounted.java rename to libs/common/src/main/java/org/opensearch/common/util/concurrent/RefCounted.java index 20a2222614343..1153df7225a05 100644 --- a/libs/core/src/main/java/org/opensearch/common/util/concurrent/RefCounted.java +++ b/libs/common/src/main/java/org/opensearch/common/util/concurrent/RefCounted.java @@ -49,6 +49,8 @@ * inst.decRef(); * } * + * + * @opensearch.api */ public interface RefCounted { diff --git a/libs/common/src/main/java/org/opensearch/common/util/concurrent/package-info.java b/libs/common/src/main/java/org/opensearch/common/util/concurrent/package-info.java new file mode 100644 index 0000000000000..b17aeac7e4c4e --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/common/util/concurrent/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Common concurrency utilities used across opensearch. */ +package org.opensearch.common.util.concurrent; diff --git a/libs/common/src/main/java/org/opensearch/common/util/package-info.java b/libs/common/src/main/java/org/opensearch/common/util/package-info.java new file mode 100644 index 0000000000000..98f786e2e82bc --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/common/util/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Common utilities used across opensearch. */ +package org.opensearch.common.util; diff --git a/libs/common/src/main/java/org/opensearch/package-info.java b/libs/common/src/main/java/org/opensearch/package-info.java new file mode 100644 index 0000000000000..6a75558502c72 --- /dev/null +++ b/libs/common/src/main/java/org/opensearch/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Base opensearch package. */ +package org.opensearch; diff --git a/libs/core/src/test/java/org/opensearch/bootstrap/JarHellTests.java b/libs/common/src/test/java/org/opensearch/bootstrap/JarHellTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/bootstrap/JarHellTests.java rename to libs/common/src/test/java/org/opensearch/bootstrap/JarHellTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/CharArraysTests.java b/libs/common/src/test/java/org/opensearch/common/CharArraysTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/CharArraysTests.java rename to libs/common/src/test/java/org/opensearch/common/CharArraysTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/SetOnceTests.java b/libs/common/src/test/java/org/opensearch/common/SetOnceTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/SetOnceTests.java rename to libs/common/src/test/java/org/opensearch/common/SetOnceTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/collect/ListTests.java b/libs/common/src/test/java/org/opensearch/common/collect/ListTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/collect/ListTests.java rename to libs/common/src/test/java/org/opensearch/common/collect/ListTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/collect/MapTests.java b/libs/common/src/test/java/org/opensearch/common/collect/MapTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/collect/MapTests.java rename to libs/common/src/test/java/org/opensearch/common/collect/MapTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/collect/SetTests.java b/libs/common/src/test/java/org/opensearch/common/collect/SetTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/collect/SetTests.java rename to libs/common/src/test/java/org/opensearch/common/collect/SetTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/collect/TupleTests.java b/libs/common/src/test/java/org/opensearch/common/collect/TupleTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/collect/TupleTests.java rename to libs/common/src/test/java/org/opensearch/common/collect/TupleTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/unit/TimeValueTests.java b/libs/common/src/test/java/org/opensearch/common/unit/TimeValueTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/unit/TimeValueTests.java rename to libs/common/src/test/java/org/opensearch/common/unit/TimeValueTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/util/OpenSearchSloppyMathTests.java b/libs/common/src/test/java/org/opensearch/common/util/OpenSearchSloppyMathTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/util/OpenSearchSloppyMathTests.java rename to libs/common/src/test/java/org/opensearch/common/util/OpenSearchSloppyMathTests.java diff --git a/libs/core/src/test/java/org/opensearch/common/util/concurrent/RefCountedTests.java b/libs/common/src/test/java/org/opensearch/common/util/concurrent/RefCountedTests.java similarity index 100% rename from libs/core/src/test/java/org/opensearch/common/util/concurrent/RefCountedTests.java rename to libs/common/src/test/java/org/opensearch/common/util/concurrent/RefCountedTests.java diff --git a/libs/core/src/test/resources/org/opensearch/bootstrap/duplicate-classes.jar b/libs/common/src/test/resources/org/opensearch/bootstrap/duplicate-classes.jar similarity index 100% rename from libs/core/src/test/resources/org/opensearch/bootstrap/duplicate-classes.jar rename to libs/common/src/test/resources/org/opensearch/bootstrap/duplicate-classes.jar diff --git a/libs/core/src/test/resources/org/opensearch/bootstrap/duplicate-xmlbeans-classes.jar b/libs/common/src/test/resources/org/opensearch/bootstrap/duplicate-xmlbeans-classes.jar similarity index 100% rename from libs/core/src/test/resources/org/opensearch/bootstrap/duplicate-xmlbeans-classes.jar rename to libs/common/src/test/resources/org/opensearch/bootstrap/duplicate-xmlbeans-classes.jar diff --git a/libs/core/build.gradle b/libs/core/build.gradle index fb8bed207dbc6..fbd8f421a08d9 100644 --- a/libs/core/build.gradle +++ b/libs/core/build.gradle @@ -75,8 +75,7 @@ if (!isEclipse) { } dependencies { - // This dependency is used only by :libs:core for null-checking interop with other tools - compileOnly "com.google.code.findbugs:jsr305:3.0.2" + api project(':libs:opensearch-common') testImplementation "com.carrotsearch.randomizedtesting:randomizedtesting-runner:${versions.randomizedrunner}" testImplementation "junit:junit:${versions.junit}" diff --git a/libs/core/src/main/java/org/opensearch/core/internal/io/IOUtils.java b/libs/core/src/main/java/org/opensearch/core/internal/io/IOUtils.java index 00ecb8b82d744..25494ca440ef4 100644 --- a/libs/core/src/main/java/org/opensearch/core/internal/io/IOUtils.java +++ b/libs/core/src/main/java/org/opensearch/core/internal/io/IOUtils.java @@ -52,6 +52,8 @@ /** * Utilities for common I/O methods. Borrowed heavily from Lucene (org.apache.lucene.util.IOUtils). + * + * @opensearch.internal */ public final class IOUtils { diff --git a/libs/core/src/main/java/org/opensearch/core/internal/io/Streams.java b/libs/core/src/main/java/org/opensearch/core/internal/io/Streams.java index 67765392b1d46..a54e23f18d1a5 100644 --- a/libs/core/src/main/java/org/opensearch/core/internal/io/Streams.java +++ b/libs/core/src/main/java/org/opensearch/core/internal/io/Streams.java @@ -42,6 +42,8 @@ *

    * Mainly for use within the framework, * but also useful for application code. + * + * @opensearch.internal */ public abstract class Streams { diff --git a/libs/core/src/main/java/org/opensearch/core/internal/io/package-info.java b/libs/core/src/main/java/org/opensearch/core/internal/io/package-info.java new file mode 100644 index 0000000000000..9efac98861647 --- /dev/null +++ b/libs/core/src/main/java/org/opensearch/core/internal/io/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Internal I/O classes - todo refactor to common */ +package org.opensearch.core.internal.io; diff --git a/libs/core/src/main/java/org/opensearch/core/internal/net/NetUtils.java b/libs/core/src/main/java/org/opensearch/core/internal/net/NetUtils.java index 6ea3b6f953c50..a77efb81e5bd0 100644 --- a/libs/core/src/main/java/org/opensearch/core/internal/net/NetUtils.java +++ b/libs/core/src/main/java/org/opensearch/core/internal/net/NetUtils.java @@ -41,6 +41,8 @@ /** * Utilities for network-related methods. + * + * @opensearch.internal */ public class NetUtils { diff --git a/libs/core/src/main/java/org/opensearch/core/internal/net/package-info.java b/libs/core/src/main/java/org/opensearch/core/internal/net/package-info.java new file mode 100644 index 0000000000000..af3e1a4503b6f --- /dev/null +++ b/libs/core/src/main/java/org/opensearch/core/internal/net/package-info.java @@ -0,0 +1,10 @@ +/* + * SPDX-License-Identifier: Apache-2.0 + * + * The OpenSearch Contributors require contributions made to + * this file be licensed under the Apache-2.0 license or a + * compatible open source license. + */ + +/** Internal Network Utility classes - todo refactor to common */ +package org.opensearch.core.internal.net; diff --git a/libs/nio/build.gradle b/libs/nio/build.gradle index cae9f7e6feb26..a497ace80f664 100644 --- a/libs/nio/build.gradle +++ b/libs/nio/build.gradle @@ -29,11 +29,12 @@ */ import org.opensearch.gradle.info.BuildParams - + apply plugin: 'opensearch.publish' dependencies { api project(':libs:opensearch-core') + api project(':libs:opensearch-common') testImplementation "com.carrotsearch.randomizedtesting:randomizedtesting-runner:${versions.randomizedrunner}" testImplementation "junit:junit:${versions.junit}" diff --git a/libs/ssl-config/build.gradle b/libs/ssl-config/build.gradle index 456641f0d7645..2ace2e03bc09d 100644 --- a/libs/ssl-config/build.gradle +++ b/libs/ssl-config/build.gradle @@ -29,10 +29,11 @@ */ import org.opensearch.gradle.info.BuildParams - + apply plugin: "opensearch.publish" dependencies { + api project(':libs:opensearch-common') api project(':libs:opensearch-core') testImplementation(project(":test:framework")) { diff --git a/libs/x-content/build.gradle b/libs/x-content/build.gradle index 3d0465783d7a0..85fc2c7e85933 100644 --- a/libs/x-content/build.gradle +++ b/libs/x-content/build.gradle @@ -32,6 +32,7 @@ apply plugin: 'opensearch.build' apply plugin: 'opensearch.publish' dependencies { + api project(':libs:opensearch-common') api project(':libs:opensearch-core') api "org.yaml:snakeyaml:${versions.snakeyaml}" diff --git a/qa/evil-tests/src/test/java/org/opensearch/common/logging/EvilLoggerTests.java b/qa/evil-tests/src/test/java/org/opensearch/common/logging/EvilLoggerTests.java index e6548885a8d10..f2015b885ad0b 100644 --- a/qa/evil-tests/src/test/java/org/opensearch/common/logging/EvilLoggerTests.java +++ b/qa/evil-tests/src/test/java/org/opensearch/common/logging/EvilLoggerTests.java @@ -41,7 +41,6 @@ import org.apache.logging.log4j.core.appender.CountingNoOpAppender; import org.apache.logging.log4j.core.config.Configurator; import org.apache.logging.log4j.message.ParameterizedMessage; -import org.apache.lucene.util.Constants; import org.opensearch.cli.UserException; import org.opensearch.cluster.ClusterName; import org.opensearch.common.Randomness; @@ -62,7 +61,6 @@ import java.util.ArrayList; import java.util.Comparator; import java.util.List; -import java.util.Set; import java.util.concurrent.BrokenBarrierException; import java.util.concurrent.CyclicBarrier; import java.util.regex.Matcher; @@ -71,7 +69,6 @@ import java.util.stream.IntStream; import static org.opensearch.common.logging.DeprecationLogger.DEPRECATION; -import static org.hamcrest.Matchers.endsWith; import static org.hamcrest.Matchers.equalTo; import static org.hamcrest.Matchers.hasItem; import static org.hamcrest.Matchers.lessThan; diff --git a/qa/os/build.gradle b/qa/os/build.gradle index 9a1e6f781faec..66c6525439dac 100644 --- a/qa/os/build.gradle +++ b/qa/os/build.gradle @@ -47,6 +47,7 @@ dependencies { api "commons-codec:commons-codec:${versions.commonscodec}" api "commons-logging:commons-logging:${versions.commonslogging}" + api project(':libs:opensearch-common') api project(':libs:opensearch-core') testImplementation "com.fasterxml.jackson.core:jackson-annotations:${versions.jackson}" diff --git a/server/build.gradle b/server/build.gradle index 5b65b90a5e902..2eea312699798 100644 --- a/server/build.gradle +++ b/server/build.gradle @@ -87,6 +87,7 @@ if (!isEclipse) { dependencies { + api project(':libs:opensearch-common') api project(':libs:opensearch-core') api project(':libs:opensearch-secure-sm') api project(':libs:opensearch-x-content') diff --git a/server/src/main/java/org/opensearch/monitor/process/ProcessStats.java b/server/src/main/java/org/opensearch/monitor/process/ProcessStats.java index 874b32a74c912..03a210df16fb5 100644 --- a/server/src/main/java/org/opensearch/monitor/process/ProcessStats.java +++ b/server/src/main/java/org/opensearch/monitor/process/ProcessStats.java @@ -37,7 +37,6 @@ import org.opensearch.common.io.stream.Writeable; import org.opensearch.common.unit.ByteSizeValue; import org.opensearch.common.unit.TimeValue; -import org.opensearch.common.xcontent.ToXContent.Params; import org.opensearch.common.xcontent.ToXContentFragment; import org.opensearch.common.xcontent.XContentBuilder; diff --git a/server/src/main/java/org/opensearch/node/AdaptiveSelectionStats.java b/server/src/main/java/org/opensearch/node/AdaptiveSelectionStats.java index 8501987e5bb8d..1da67080329d5 100644 --- a/server/src/main/java/org/opensearch/node/AdaptiveSelectionStats.java +++ b/server/src/main/java/org/opensearch/node/AdaptiveSelectionStats.java @@ -37,7 +37,6 @@ import org.opensearch.common.io.stream.Writeable; import org.opensearch.common.unit.TimeValue; import org.opensearch.common.util.set.Sets; -import org.opensearch.common.xcontent.ToXContent.Params; import org.opensearch.common.xcontent.ToXContentFragment; import org.opensearch.common.xcontent.XContentBuilder; diff --git a/server/src/main/java/org/opensearch/search/SearchService.java b/server/src/main/java/org/opensearch/search/SearchService.java index c0d63fd89838d..a15489cd4d127 100644 --- a/server/src/main/java/org/opensearch/search/SearchService.java +++ b/server/src/main/java/org/opensearch/search/SearchService.java @@ -41,12 +41,9 @@ import org.opensearch.action.ActionRunnable; import org.opensearch.action.OriginalIndices; import org.opensearch.action.search.DeletePitInfo; -import org.opensearch.action.search.DeletePitInfo; -import org.opensearch.action.search.DeletePitResponse; import org.opensearch.action.search.DeletePitResponse; import org.opensearch.action.search.ListPitInfo; import org.opensearch.action.search.PitSearchContextIdForNode; -import org.opensearch.action.search.PitSearchContextIdForNode; import org.opensearch.action.search.SearchRequest; import org.opensearch.action.search.SearchShardTask; import org.opensearch.action.search.SearchType; @@ -145,7 +142,6 @@ import java.io.IOException; import java.util.ArrayList; -import java.util.ArrayList; import java.util.Collections; import java.util.HashMap; import java.util.List; From 8d5d2e6b10db66784279c72715fb643f0fed38d0 Mon Sep 17 00:00:00 2001 From: Nick Knize Date: Tue, 31 Jan 2023 10:38:27 -0600 Subject: [PATCH 19/20] [Upgrade] Lucene 9.5.0 release (#6078) Upgrades to official release of Lucene 9.5. Signed-off-by: Nicholas Walter Knize --- CHANGELOG.md | 1 + buildSrc/version.properties | 2 +- .../common/HighlighterWithAnalyzersTests.java | 12 ++++++++++-- ...e-expressions-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../licenses/lucene-expressions-9.5.0.jar.sha1 | 1 + ...-analysis-icu-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../licenses/lucene-analysis-icu-9.5.0.jar.sha1 | 1 + ...ysis-kuromoji-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../lucene-analysis-kuromoji-9.5.0.jar.sha1 | 1 + ...analysis-nori-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../licenses/lucene-analysis-nori-9.5.0.jar.sha1 | 1 + ...ysis-phonetic-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../lucene-analysis-phonetic-9.5.0.jar.sha1 | 1 + ...lysis-smartcn-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../lucene-analysis-smartcn-9.5.0.jar.sha1 | 1 + ...lysis-stempel-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../lucene-analysis-stempel-9.5.0.jar.sha1 | 1 + ...is-morfologik-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../lucene-analysis-morfologik-9.5.0.jar.sha1 | 1 + ...alysis-common-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../lucene-analysis-common-9.5.0.jar.sha1 | 1 + ...ckward-codecs-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../lucene-backward-codecs-9.5.0.jar.sha1 | 1 + .../lucene-core-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-core-9.5.0.jar.sha1 | 1 + ...cene-grouping-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-grouping-9.5.0.jar.sha1 | 1 + ...e-highlighter-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../licenses/lucene-highlighter-9.5.0.jar.sha1 | 1 + .../lucene-join-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-join-9.5.0.jar.sha1 | 1 + ...lucene-memory-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-memory-9.5.0.jar.sha1 | 1 + .../lucene-misc-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-misc-9.5.0.jar.sha1 | 1 + ...ucene-queries-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-queries-9.5.0.jar.sha1 | 1 + ...e-queryparser-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../licenses/lucene-queryparser-9.5.0.jar.sha1 | 1 + ...ucene-sandbox-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-sandbox-9.5.0.jar.sha1 | 1 + ...patial-extras-9.5.0-snapshot-0878271.jar.sha1 | 1 - .../lucene-spatial-extras-9.5.0.jar.sha1 | 1 + ...ene-spatial3d-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-spatial3d-9.5.0.jar.sha1 | 1 + ...ucene-suggest-9.5.0-snapshot-0878271.jar.sha1 | 1 - server/licenses/lucene-suggest-9.5.0.jar.sha1 | 1 + .../lucene/search/MultiPhrasePrefixQuery.java | 16 +++++++++++++++- .../index/engine/TranslogLeafReader.java | 14 ++++++++++---- .../opensearch/index/mapper/DateFieldMapper.java | 2 +- .../index/mapper/NumberFieldMapper.java | 2 +- .../index/mapper/DateFieldTypeTests.java | 2 +- .../index/mapper/NumberFieldTypeTests.java | 2 +- 53 files changed, 63 insertions(+), 34 deletions(-) delete mode 100644 modules/lang-expression/licenses/lucene-expressions-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 modules/lang-expression/licenses/lucene-expressions-9.5.0.jar.sha1 delete mode 100644 plugins/analysis-icu/licenses/lucene-analysis-icu-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 plugins/analysis-icu/licenses/lucene-analysis-icu-9.5.0.jar.sha1 delete mode 100644 plugins/analysis-kuromoji/licenses/lucene-analysis-kuromoji-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 plugins/analysis-kuromoji/licenses/lucene-analysis-kuromoji-9.5.0.jar.sha1 delete mode 100644 plugins/analysis-nori/licenses/lucene-analysis-nori-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 plugins/analysis-nori/licenses/lucene-analysis-nori-9.5.0.jar.sha1 delete mode 100644 plugins/analysis-phonetic/licenses/lucene-analysis-phonetic-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 plugins/analysis-phonetic/licenses/lucene-analysis-phonetic-9.5.0.jar.sha1 delete mode 100644 plugins/analysis-smartcn/licenses/lucene-analysis-smartcn-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 plugins/analysis-smartcn/licenses/lucene-analysis-smartcn-9.5.0.jar.sha1 delete mode 100644 plugins/analysis-stempel/licenses/lucene-analysis-stempel-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 plugins/analysis-stempel/licenses/lucene-analysis-stempel-9.5.0.jar.sha1 delete mode 100644 plugins/analysis-ukrainian/licenses/lucene-analysis-morfologik-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 plugins/analysis-ukrainian/licenses/lucene-analysis-morfologik-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-analysis-common-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-analysis-common-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-backward-codecs-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-backward-codecs-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-core-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-core-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-grouping-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-grouping-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-highlighter-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-highlighter-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-join-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-join-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-memory-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-memory-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-misc-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-misc-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-queries-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-queries-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-queryparser-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-queryparser-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-sandbox-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-sandbox-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-spatial-extras-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-spatial-extras-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-spatial3d-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-spatial3d-9.5.0.jar.sha1 delete mode 100644 server/licenses/lucene-suggest-9.5.0-snapshot-0878271.jar.sha1 create mode 100644 server/licenses/lucene-suggest-9.5.0.jar.sha1 diff --git a/CHANGELOG.md b/CHANGELOG.md index 2d3c8fa81cc75..3d371b56e930b 100644 --- a/CHANGELOG.md +++ b/CHANGELOG.md @@ -89,6 +89,7 @@ The format is based on [Keep a Changelog](https://keepachangelog.com/en/1.0.0/), - OpenJDK Update (January 2023 Patch releases) ([#6074](https://github.com/opensearch-project/OpenSearch/pull/6074)) - Bumps `Mockito` from 4.7.0 to 5.1.0, `ByteBuddy` from 1.12.18 to 1.12.22 ([#6076](https://github.com/opensearch-project/OpenSearch/pull/6076)) - Bumps `joda` from 2.10.13 to 2.12.2 ([#6083](https://github.com/opensearch-project/OpenSearch/pull/6083)) +- Upgrade to Lucene 9.5.0 ([#5878](https://github.com/opensearch-project/OpenSearch/pull/6078)) ### Changed - Use ReplicationFailedException instead of OpensearchException in ReplicationTarget ([#4725](https://github.com/opensearch-project/OpenSearch/pull/4725)) diff --git a/buildSrc/version.properties b/buildSrc/version.properties index 58fb0074d11f2..2599c38dbcb2f 100644 --- a/buildSrc/version.properties +++ b/buildSrc/version.properties @@ -1,5 +1,5 @@ opensearch = 3.0.0 -lucene = 9.5.0-snapshot-0878271 +lucene = 9.5.0 bundled_jdk_vendor = adoptium bundled_jdk = 19.0.2+7 diff --git a/modules/analysis-common/src/test/java/org/opensearch/analysis/common/HighlighterWithAnalyzersTests.java b/modules/analysis-common/src/test/java/org/opensearch/analysis/common/HighlighterWithAnalyzersTests.java index 74ed3cd79e753..8d39258d31cd6 100644 --- a/modules/analysis-common/src/test/java/org/opensearch/analysis/common/HighlighterWithAnalyzersTests.java +++ b/modules/analysis-common/src/test/java/org/opensearch/analysis/common/HighlighterWithAnalyzersTests.java @@ -50,6 +50,7 @@ import static org.opensearch.client.Requests.searchRequest; import static org.opensearch.common.xcontent.XContentFactory.jsonBuilder; import static org.opensearch.index.query.QueryBuilders.boolQuery; +import static org.opensearch.index.query.QueryBuilders.matchAllQuery; import static org.opensearch.index.query.QueryBuilders.matchPhrasePrefixQuery; import static org.opensearch.index.query.QueryBuilders.matchPhraseQuery; import static org.opensearch.index.query.QueryBuilders.matchQuery; @@ -61,6 +62,7 @@ import static org.hamcrest.Matchers.anyOf; import static org.hamcrest.Matchers.equalTo; import static org.hamcrest.Matchers.startsWith; +import static org.opensearch.test.hamcrest.OpenSearchAssertions.assertHitCount; public class HighlighterWithAnalyzersTests extends OpenSearchIntegTestCase { @Override @@ -270,9 +272,13 @@ public void testPhrasePrefix() throws IOException { refresh(); logger.info("--> highlighting and searching on field0"); - SearchSourceBuilder source = searchSource().query(matchPhrasePrefixQuery("field0", "bro")) - .highlighter(highlight().field("field0").order("score").preTags("").postTags("")); + SearchSourceBuilder source = searchSource().query(matchAllQuery()); SearchResponse searchResponse = client().search(searchRequest("first_test_index").source(source)).actionGet(); + assertHitCount(searchResponse, 2); + + source = searchSource().query(matchPhrasePrefixQuery("field0", "bro")) + .highlighter(highlight().field("field0").order("score").preTags("").postTags("")); + searchResponse = client().search(searchRequest("first_test_index").source(source)).actionGet(); assertHighlight(searchResponse, 0, "field0", 0, 1, equalTo("The quick brown fox jumps over the lazy dog")); @@ -415,6 +421,7 @@ public void testPhrasePrefix() throws IOException { public static XContentBuilder type1TermVectorMapping() throws IOException { return XContentFactory.jsonBuilder() .startObject() + .startObject("_doc") .startObject("properties") .startObject("field1") .field("type", "text") @@ -425,6 +432,7 @@ public static XContentBuilder type1TermVectorMapping() throws IOException { .field("term_vector", "with_positions_offsets") .endObject() .endObject() + .endObject() .endObject(); } } diff --git a/modules/lang-expression/licenses/lucene-expressions-9.5.0-snapshot-0878271.jar.sha1 b/modules/lang-expression/licenses/lucene-expressions-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 7b246b88ba5a8..0000000000000 --- a/modules/lang-expression/licenses/lucene-expressions-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -6912f9f9ceebbe5a54795035984709a4ada115e8 \ No newline at end of file diff --git a/modules/lang-expression/licenses/lucene-expressions-9.5.0.jar.sha1 b/modules/lang-expression/licenses/lucene-expressions-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..c1340a55815e5 --- /dev/null +++ b/modules/lang-expression/licenses/lucene-expressions-9.5.0.jar.sha1 @@ -0,0 +1 @@ +62f3d3630ecc14069d5c24b9693df5a2787f8202 \ No newline at end of file diff --git a/plugins/analysis-icu/licenses/lucene-analysis-icu-9.5.0-snapshot-0878271.jar.sha1 b/plugins/analysis-icu/licenses/lucene-analysis-icu-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index e906bd304e64f..0000000000000 --- a/plugins/analysis-icu/licenses/lucene-analysis-icu-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -477512b093aa1dd55ea9340c3a16128f6722161b \ No newline at end of file diff --git a/plugins/analysis-icu/licenses/lucene-analysis-icu-9.5.0.jar.sha1 b/plugins/analysis-icu/licenses/lucene-analysis-icu-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..1d3eb41aab0a1 --- /dev/null +++ b/plugins/analysis-icu/licenses/lucene-analysis-icu-9.5.0.jar.sha1 @@ -0,0 +1 @@ +11c816250e4ff106151fd8cb69e61ead4fb4a8dd \ No newline at end of file diff --git a/plugins/analysis-kuromoji/licenses/lucene-analysis-kuromoji-9.5.0-snapshot-0878271.jar.sha1 b/plugins/analysis-kuromoji/licenses/lucene-analysis-kuromoji-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index fcdd55305387f..0000000000000 --- a/plugins/analysis-kuromoji/licenses/lucene-analysis-kuromoji-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -89e9389e166bbd5e774d4b62acd75bf37481062e \ No newline at end of file diff --git a/plugins/analysis-kuromoji/licenses/lucene-analysis-kuromoji-9.5.0.jar.sha1 b/plugins/analysis-kuromoji/licenses/lucene-analysis-kuromoji-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..31f1eb3991d5a --- /dev/null +++ b/plugins/analysis-kuromoji/licenses/lucene-analysis-kuromoji-9.5.0.jar.sha1 @@ -0,0 +1 @@ +4555fc64ce9d63c6c1f4983e816526f896606e9f \ No newline at end of file diff --git a/plugins/analysis-nori/licenses/lucene-analysis-nori-9.5.0-snapshot-0878271.jar.sha1 b/plugins/analysis-nori/licenses/lucene-analysis-nori-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index d1a86508da090..0000000000000 --- a/plugins/analysis-nori/licenses/lucene-analysis-nori-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -928c3eb74edf7ab62ea7d0948703f2867299cb56 \ No newline at end of file diff --git a/plugins/analysis-nori/licenses/lucene-analysis-nori-9.5.0.jar.sha1 b/plugins/analysis-nori/licenses/lucene-analysis-nori-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..47bfe0e4b667d --- /dev/null +++ b/plugins/analysis-nori/licenses/lucene-analysis-nori-9.5.0.jar.sha1 @@ -0,0 +1 @@ +2fd55607da8adf8234169d9f4120119c05a8175c \ No newline at end of file diff --git a/plugins/analysis-phonetic/licenses/lucene-analysis-phonetic-9.5.0-snapshot-0878271.jar.sha1 b/plugins/analysis-phonetic/licenses/lucene-analysis-phonetic-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 8b8986ee07ab2..0000000000000 --- a/plugins/analysis-phonetic/licenses/lucene-analysis-phonetic-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -2d5640fe3a99b9af4c469243032b89dba32f7ab2 \ No newline at end of file diff --git a/plugins/analysis-phonetic/licenses/lucene-analysis-phonetic-9.5.0.jar.sha1 b/plugins/analysis-phonetic/licenses/lucene-analysis-phonetic-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..99ce9a0a4cb9a --- /dev/null +++ b/plugins/analysis-phonetic/licenses/lucene-analysis-phonetic-9.5.0.jar.sha1 @@ -0,0 +1 @@ +c97163035f1d46249505d15fbdc427b6ae03549a \ No newline at end of file diff --git a/plugins/analysis-smartcn/licenses/lucene-analysis-smartcn-9.5.0-snapshot-0878271.jar.sha1 b/plugins/analysis-smartcn/licenses/lucene-analysis-smartcn-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 20f4c39f40ebf..0000000000000 --- a/plugins/analysis-smartcn/licenses/lucene-analysis-smartcn-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -44c3ceefd4999b046b935d8737c26c1a835c8d5a \ No newline at end of file diff --git a/plugins/analysis-smartcn/licenses/lucene-analysis-smartcn-9.5.0.jar.sha1 b/plugins/analysis-smartcn/licenses/lucene-analysis-smartcn-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..e63db9c407a77 --- /dev/null +++ b/plugins/analysis-smartcn/licenses/lucene-analysis-smartcn-9.5.0.jar.sha1 @@ -0,0 +1 @@ +a08ae3e2212837c98d5e1174cf1a293a5c8c0373 \ No newline at end of file diff --git a/plugins/analysis-stempel/licenses/lucene-analysis-stempel-9.5.0-snapshot-0878271.jar.sha1 b/plugins/analysis-stempel/licenses/lucene-analysis-stempel-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 899575f029626..0000000000000 --- a/plugins/analysis-stempel/licenses/lucene-analysis-stempel-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -6da2070b19949a397e45f2b54a739cef403d0c94 \ No newline at end of file diff --git a/plugins/analysis-stempel/licenses/lucene-analysis-stempel-9.5.0.jar.sha1 b/plugins/analysis-stempel/licenses/lucene-analysis-stempel-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..d2f956a296f96 --- /dev/null +++ b/plugins/analysis-stempel/licenses/lucene-analysis-stempel-9.5.0.jar.sha1 @@ -0,0 +1 @@ +483906d2ebad0ff910d52382c2f027b511788b58 \ No newline at end of file diff --git a/plugins/analysis-ukrainian/licenses/lucene-analysis-morfologik-9.5.0-snapshot-0878271.jar.sha1 b/plugins/analysis-ukrainian/licenses/lucene-analysis-morfologik-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 49000d3e4b151..0000000000000 --- a/plugins/analysis-ukrainian/licenses/lucene-analysis-morfologik-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -1803e928dd330599048b71386456fb90acbed5c1 \ No newline at end of file diff --git a/plugins/analysis-ukrainian/licenses/lucene-analysis-morfologik-9.5.0.jar.sha1 b/plugins/analysis-ukrainian/licenses/lucene-analysis-morfologik-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..d28a2fa06fe28 --- /dev/null +++ b/plugins/analysis-ukrainian/licenses/lucene-analysis-morfologik-9.5.0.jar.sha1 @@ -0,0 +1 @@ +cb00cd193e2f927e6357cf33e0880034571ea528 \ No newline at end of file diff --git a/server/licenses/lucene-analysis-common-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-analysis-common-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index de9ad9baf756b..0000000000000 --- a/server/licenses/lucene-analysis-common-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -92c380f063c5b75efd6b3c48dffc942bfe21360f \ No newline at end of file diff --git a/server/licenses/lucene-analysis-common-9.5.0.jar.sha1 b/server/licenses/lucene-analysis-common-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..bcd78036b0c99 --- /dev/null +++ b/server/licenses/lucene-analysis-common-9.5.0.jar.sha1 @@ -0,0 +1 @@ +f68660102455a466f98cac0501723bed7e7c6407 \ No newline at end of file diff --git a/server/licenses/lucene-backward-codecs-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-backward-codecs-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 77aa7df611354..0000000000000 --- a/server/licenses/lucene-backward-codecs-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -2f0f211b1ed0447135f84a36935625e9a33a98e5 \ No newline at end of file diff --git a/server/licenses/lucene-backward-codecs-9.5.0.jar.sha1 b/server/licenses/lucene-backward-codecs-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..4000ee7fc0e93 --- /dev/null +++ b/server/licenses/lucene-backward-codecs-9.5.0.jar.sha1 @@ -0,0 +1 @@ +66be22239d5058b0b8a8aeba03dc047a276efafa \ No newline at end of file diff --git a/server/licenses/lucene-core-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-core-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index f10ba49ec6dd7..0000000000000 --- a/server/licenses/lucene-core-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -1f8895d068b5d98c16d48f267a772d0243148040 \ No newline at end of file diff --git a/server/licenses/lucene-core-9.5.0.jar.sha1 b/server/licenses/lucene-core-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..bca72d4375dab --- /dev/null +++ b/server/licenses/lucene-core-9.5.0.jar.sha1 @@ -0,0 +1 @@ +bba4ba5d30e71a5f0017e45e8469db8cff8ad102 \ No newline at end of file diff --git a/server/licenses/lucene-grouping-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-grouping-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index ab153ccb80f0b..0000000000000 --- a/server/licenses/lucene-grouping-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -f12819e51f145624ef7b1fd93aaa543cec13e3aa \ No newline at end of file diff --git a/server/licenses/lucene-grouping-9.5.0.jar.sha1 b/server/licenses/lucene-grouping-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..08ed52d3379ec --- /dev/null +++ b/server/licenses/lucene-grouping-9.5.0.jar.sha1 @@ -0,0 +1 @@ +b15fe0a55a82168c810d0447ec2e244d16d94f01 \ No newline at end of file diff --git a/server/licenses/lucene-highlighter-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-highlighter-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 0885f1fda24bf..0000000000000 --- a/server/licenses/lucene-highlighter-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -ed88cca26580a32234e3f605cf963703ea99eb60 \ No newline at end of file diff --git a/server/licenses/lucene-highlighter-9.5.0.jar.sha1 b/server/licenses/lucene-highlighter-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..0df6615e43c66 --- /dev/null +++ b/server/licenses/lucene-highlighter-9.5.0.jar.sha1 @@ -0,0 +1 @@ +8b5e3cea3370838bda1d730cf55176c24a763d2e \ No newline at end of file diff --git a/server/licenses/lucene-join-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-join-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index afdba4c231926..0000000000000 --- a/server/licenses/lucene-join-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -f51934b2362e827d9f467507624b773153f0ca01 \ No newline at end of file diff --git a/server/licenses/lucene-join-9.5.0.jar.sha1 b/server/licenses/lucene-join-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..d441e3215820a --- /dev/null +++ b/server/licenses/lucene-join-9.5.0.jar.sha1 @@ -0,0 +1 @@ +85a4208614a5660297effce441883687b010073b \ No newline at end of file diff --git a/server/licenses/lucene-memory-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-memory-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 29d894a978024..0000000000000 --- a/server/licenses/lucene-memory-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -54744097c6882a498c08ffd8a42b54da00c9420c \ No newline at end of file diff --git a/server/licenses/lucene-memory-9.5.0.jar.sha1 b/server/licenses/lucene-memory-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..0fb289af21987 --- /dev/null +++ b/server/licenses/lucene-memory-9.5.0.jar.sha1 @@ -0,0 +1 @@ +d632e63d08837be715046c8ccb4fb804acd3d7e4 \ No newline at end of file diff --git a/server/licenses/lucene-misc-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-misc-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index b09c965d042b3..0000000000000 --- a/server/licenses/lucene-misc-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -b9061dea1178e0cac86147d7b69fc53bf2f8ee58 \ No newline at end of file diff --git a/server/licenses/lucene-misc-9.5.0.jar.sha1 b/server/licenses/lucene-misc-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..985fbb18ea6d0 --- /dev/null +++ b/server/licenses/lucene-misc-9.5.0.jar.sha1 @@ -0,0 +1 @@ +64773801b8ba8141f4256d22da598de40d6f3033 \ No newline at end of file diff --git a/server/licenses/lucene-queries-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-queries-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 604edd3b00842..0000000000000 --- a/server/licenses/lucene-queries-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -eca07717c0ee563c5337222fd1a1b7ef5f03f34f \ No newline at end of file diff --git a/server/licenses/lucene-queries-9.5.0.jar.sha1 b/server/licenses/lucene-queries-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..4ac6d8161fc58 --- /dev/null +++ b/server/licenses/lucene-queries-9.5.0.jar.sha1 @@ -0,0 +1 @@ +ae7930fa0ea91198905d695a222ef7f09de1c2dd \ No newline at end of file diff --git a/server/licenses/lucene-queryparser-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-queryparser-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 11ba3ca14d028..0000000000000 --- a/server/licenses/lucene-queryparser-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -4d1ed21bc62940c4545778dc5f6249b67d08b095 \ No newline at end of file diff --git a/server/licenses/lucene-queryparser-9.5.0.jar.sha1 b/server/licenses/lucene-queryparser-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..45b6229e713c2 --- /dev/null +++ b/server/licenses/lucene-queryparser-9.5.0.jar.sha1 @@ -0,0 +1 @@ +b3e3d9434bcfce242e242364c64eab0888d285e2 \ No newline at end of file diff --git a/server/licenses/lucene-sandbox-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-sandbox-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 6bbb3be0bd801..0000000000000 --- a/server/licenses/lucene-sandbox-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -c564c73080cff226a131847b4361bc9c2155f00c \ No newline at end of file diff --git a/server/licenses/lucene-sandbox-9.5.0.jar.sha1 b/server/licenses/lucene-sandbox-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..d43ebd2d27119 --- /dev/null +++ b/server/licenses/lucene-sandbox-9.5.0.jar.sha1 @@ -0,0 +1 @@ +408be287b0f421cf8afd655579a0cc65aba61b28 \ No newline at end of file diff --git a/server/licenses/lucene-spatial-extras-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-spatial-extras-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index a4def52c3788e..0000000000000 --- a/server/licenses/lucene-spatial-extras-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -7a654fcc671e0372a9becd7c7cbd9a1f43569106 \ No newline at end of file diff --git a/server/licenses/lucene-spatial-extras-9.5.0.jar.sha1 b/server/licenses/lucene-spatial-extras-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..b44f49f3601ba --- /dev/null +++ b/server/licenses/lucene-spatial-extras-9.5.0.jar.sha1 @@ -0,0 +1 @@ +4960c09365d20f024bf668f60d7c3b4e54f03750 \ No newline at end of file diff --git a/server/licenses/lucene-spatial3d-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-spatial3d-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 9b782e4eda598..0000000000000 --- a/server/licenses/lucene-spatial3d-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -33c9c43eac6b4924155eeff03d2231c2bed9b169 \ No newline at end of file diff --git a/server/licenses/lucene-spatial3d-9.5.0.jar.sha1 b/server/licenses/lucene-spatial3d-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..369e569ee754f --- /dev/null +++ b/server/licenses/lucene-spatial3d-9.5.0.jar.sha1 @@ -0,0 +1 @@ +01ed55dd384e654076b76d7083feee6ac3efe02b \ No newline at end of file diff --git a/server/licenses/lucene-suggest-9.5.0-snapshot-0878271.jar.sha1 b/server/licenses/lucene-suggest-9.5.0-snapshot-0878271.jar.sha1 deleted file mode 100644 index 5e3e9cebdc35c..0000000000000 --- a/server/licenses/lucene-suggest-9.5.0-snapshot-0878271.jar.sha1 +++ /dev/null @@ -1 +0,0 @@ -95a69fd51377d964d16057a4a9501665a2ca9a7a \ No newline at end of file diff --git a/server/licenses/lucene-suggest-9.5.0.jar.sha1 b/server/licenses/lucene-suggest-9.5.0.jar.sha1 new file mode 100644 index 0000000000000..c44ac0ec028f4 --- /dev/null +++ b/server/licenses/lucene-suggest-9.5.0.jar.sha1 @@ -0,0 +1 @@ +195624eb3a09b3dc37cc5fa51ae2f7bd61553985 \ No newline at end of file diff --git a/server/src/main/java/org/opensearch/common/lucene/search/MultiPhrasePrefixQuery.java b/server/src/main/java/org/opensearch/common/lucene/search/MultiPhrasePrefixQuery.java index 6986bd8504f84..a1ab761ec8034 100644 --- a/server/src/main/java/org/opensearch/common/lucene/search/MultiPhrasePrefixQuery.java +++ b/server/src/main/java/org/opensearch/common/lucene/search/MultiPhrasePrefixQuery.java @@ -325,6 +325,20 @@ private boolean termArraysEquals(List termArrays1, List termArra @Override public void visit(QueryVisitor visitor) { - visitor.visitLeaf(this); + if (visitor.acceptField(field)) { + // set by the subvisitor because the default returns, *this*, unless MUST_NOT is used which returns + // an ideal EMPTY_VISITOR + visitor = visitor.getSubVisitor(BooleanClause.Occur.MUST, this); + for (int i = 0; i < termArrays.size() - 1; ++i) { + if (termArrays.get(i).length == 1) { + visitor.consumeTerms(this, termArrays.get(i)[0]); // use a must if we only have a single phrase + } else { + // if more than one sub-phrase + // we should match at least one + QueryVisitor subVisitor = visitor.getSubVisitor(BooleanClause.Occur.SHOULD, this); + subVisitor.consumeTerms(this, termArrays.get(i)); + } + } + } } } diff --git a/server/src/main/java/org/opensearch/index/engine/TranslogLeafReader.java b/server/src/main/java/org/opensearch/index/engine/TranslogLeafReader.java index 258674a096e52..5efafb562df74 100644 --- a/server/src/main/java/org/opensearch/index/engine/TranslogLeafReader.java +++ b/server/src/main/java/org/opensearch/index/engine/TranslogLeafReader.java @@ -32,10 +32,12 @@ package org.opensearch.index.engine; import org.apache.lucene.index.BinaryDocValues; +import org.apache.lucene.index.ByteVectorValues; import org.apache.lucene.index.DocValuesType; import org.apache.lucene.index.FieldInfo; import org.apache.lucene.index.FieldInfos; import org.apache.lucene.index.Fields; +import org.apache.lucene.index.FloatVectorValues; import org.apache.lucene.index.IndexOptions; import org.apache.lucene.index.LeafMetaData; import org.apache.lucene.index.LeafReader; @@ -50,7 +52,6 @@ import org.apache.lucene.index.Terms; import org.apache.lucene.index.VectorEncoding; import org.apache.lucene.index.VectorSimilarityFunction; -import org.apache.lucene.index.VectorValues; import org.apache.lucene.search.TopDocs; import org.apache.lucene.util.Bits; import org.apache.lucene.util.BytesRef; @@ -255,8 +256,13 @@ public CacheHelper getReaderCacheHelper() { } @Override - public VectorValues getVectorValues(String field) throws IOException { - return getVectorValues(field); + public FloatVectorValues getFloatVectorValues(String field) throws IOException { + return getFloatVectorValues(field); + } + + @Override + public ByteVectorValues getByteVectorValues(String field) throws IOException { + return getByteVectorValues(field); } @Override @@ -265,7 +271,7 @@ public TopDocs searchNearestVectors(String field, float[] target, int k, Bits ac } @Override - public TopDocs searchNearestVectors(String field, BytesRef target, int k, Bits acceptDocs, int visitedLimit) throws IOException { + public TopDocs searchNearestVectors(String field, byte[] target, int k, Bits acceptDocs, int visitedLimit) throws IOException { throw new UnsupportedOperationException(); } } diff --git a/server/src/main/java/org/opensearch/index/mapper/DateFieldMapper.java b/server/src/main/java/org/opensearch/index/mapper/DateFieldMapper.java index 4b19fe4c5de79..1669c14d00c2a 100644 --- a/server/src/main/java/org/opensearch/index/mapper/DateFieldMapper.java +++ b/server/src/main/java/org/opensearch/index/mapper/DateFieldMapper.java @@ -37,7 +37,7 @@ import org.apache.lucene.document.StoredField; import org.apache.lucene.index.IndexReader; import org.apache.lucene.index.PointValues; -import org.apache.lucene.sandbox.search.IndexSortSortedNumericDocValuesRangeQuery; +import org.apache.lucene.search.IndexSortSortedNumericDocValuesRangeQuery; import org.apache.lucene.search.BoostQuery; import org.apache.lucene.search.IndexOrDocValuesQuery; import org.apache.lucene.search.Query; diff --git a/server/src/main/java/org/opensearch/index/mapper/NumberFieldMapper.java b/server/src/main/java/org/opensearch/index/mapper/NumberFieldMapper.java index 92b970895837f..449d6569ef700 100644 --- a/server/src/main/java/org/opensearch/index/mapper/NumberFieldMapper.java +++ b/server/src/main/java/org/opensearch/index/mapper/NumberFieldMapper.java @@ -42,7 +42,7 @@ import org.apache.lucene.document.SortedNumericDocValuesField; import org.apache.lucene.document.StoredField; import org.apache.lucene.sandbox.document.HalfFloatPoint; -import org.apache.lucene.sandbox.search.IndexSortSortedNumericDocValuesRangeQuery; +import org.apache.lucene.search.IndexSortSortedNumericDocValuesRangeQuery; import org.apache.lucene.search.BoostQuery; import org.apache.lucene.search.IndexOrDocValuesQuery; import org.apache.lucene.search.MatchNoDocsQuery; diff --git a/server/src/test/java/org/opensearch/index/mapper/DateFieldTypeTests.java b/server/src/test/java/org/opensearch/index/mapper/DateFieldTypeTests.java index 490966c5c48e5..2300d3cceab65 100644 --- a/server/src/test/java/org/opensearch/index/mapper/DateFieldTypeTests.java +++ b/server/src/test/java/org/opensearch/index/mapper/DateFieldTypeTests.java @@ -40,7 +40,7 @@ import org.apache.lucene.index.IndexWriterConfig; import org.apache.lucene.index.MultiReader; import org.apache.lucene.index.SortedNumericDocValues; -import org.apache.lucene.sandbox.search.IndexSortSortedNumericDocValuesRangeQuery; +import org.apache.lucene.search.IndexSortSortedNumericDocValuesRangeQuery; import org.apache.lucene.search.DocIdSetIterator; import org.apache.lucene.search.IndexOrDocValuesQuery; import org.apache.lucene.search.Query; diff --git a/server/src/test/java/org/opensearch/index/mapper/NumberFieldTypeTests.java b/server/src/test/java/org/opensearch/index/mapper/NumberFieldTypeTests.java index 868261130135c..ca5e8d0c6e08d 100644 --- a/server/src/test/java/org/opensearch/index/mapper/NumberFieldTypeTests.java +++ b/server/src/test/java/org/opensearch/index/mapper/NumberFieldTypeTests.java @@ -43,7 +43,7 @@ import org.apache.lucene.index.IndexWriter; import org.apache.lucene.index.IndexWriterConfig; import org.apache.lucene.sandbox.document.HalfFloatPoint; -import org.apache.lucene.sandbox.search.IndexSortSortedNumericDocValuesRangeQuery; +import org.apache.lucene.search.IndexSortSortedNumericDocValuesRangeQuery; import org.apache.lucene.search.IndexOrDocValuesQuery; import org.apache.lucene.search.IndexSearcher; import org.apache.lucene.search.MatchNoDocsQuery; From d96abfadb25a155cd119c915d7cdf7492b27d378 Mon Sep 17 00:00:00 2001 From: Suraj Singh Date: Tue, 31 Jan 2023 10:34:05 -0800 Subject: [PATCH 20/20] [Segment Replication] Mute SegmentRelocationITs failing/flaky on CI (#6113) Signed-off-by: Suraj Singh --- .../indices/replication/SegmentReplicationRelocationIT.java | 3 +++ 1 file changed, 3 insertions(+) diff --git a/server/src/internalClusterTest/java/org/opensearch/indices/replication/SegmentReplicationRelocationIT.java b/server/src/internalClusterTest/java/org/opensearch/indices/replication/SegmentReplicationRelocationIT.java index 9cffe8c056f8a..681150fb2bd83 100644 --- a/server/src/internalClusterTest/java/org/opensearch/indices/replication/SegmentReplicationRelocationIT.java +++ b/server/src/internalClusterTest/java/org/opensearch/indices/replication/SegmentReplicationRelocationIT.java @@ -55,6 +55,7 @@ private void createIndex(int replicaCount) { * This test verifies happy path when primary shard is relocated newly added node (target) in the cluster. Before * relocation and after relocation documents are indexed and documents are verified */ + @AwaitsFix(bugUrl = "https://github.com/opensearch-project/OpenSearch/issues/5669") public void testPrimaryRelocation() throws Exception { final String oldPrimary = internalCluster().startNode(); createIndex(1); @@ -131,6 +132,7 @@ public void testPrimaryRelocation() throws Exception { * failure, more documents are ingested and verified on replica; which confirms older primary still refreshing the * replicas. */ + @AwaitsFix(bugUrl = "https://github.com/opensearch-project/OpenSearch/issues/5669") public void testPrimaryRelocationWithSegRepFailure() throws Exception { final String oldPrimary = internalCluster().startNode(); createIndex(1); @@ -292,6 +294,7 @@ public void testRelocateWhileContinuouslyIndexingAndWaitingForRefresh() throws E * operations during handoff. The test verifies all docs ingested are searchable on new primary. * */ + @AwaitsFix(bugUrl = "https://github.com/opensearch-project/OpenSearch/issues/5669") public void testRelocateWithQueuedOperationsDuringHandoff() throws Exception { final String primary = internalCluster().startNode(); createIndex(1);