Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -136,6 +136,22 @@ public Void call(Object... names) {
return null;
}
});
// Add a closure to allow projects to optionally call `forbidSleep()` which will add the signatures
// to forbid all usages of `Thread.sleep`
ext.set("forbidSleep", new Closure<Void>(t) {
@Override
public Void call(Object... unused) {
final List<String> signatures = new ArrayList<>();
signatures.addAll(t.getSignatures());
signatures.add(
"java.lang.Thread#sleep(**) @ Fixed sleeps lead to non-deterministic test failures."
+ " Poll for whatever condition you're waiting for."
+ " Use helpers like `assertBusy` or the awaitility lib."
);
t.setSignatures(signatures);
return null;
}
});
// Use of the deprecated security manager APIs are pervasive so set them to warn
// globally for all projects. Replacements for (most of) these APIs are available
// so usages can move to the non-deprecated variants to avoid the warnings.
Expand Down
2 changes: 2 additions & 0 deletions server/build.gradle
Original file line number Diff line number Diff line change
Expand Up @@ -169,6 +169,8 @@ tasks.named("testingConventions").configure {
}
}

tasks.named('forbiddenApisInternalClusterTest').configure { forbidSleep() }

// Set to current version by default
def japicmpCompareTarget = System.getProperty("japicmp.compare.version")
if (japicmpCompareTarget == null) { /* use latest released version */
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -95,27 +95,13 @@ public void testCatShardsWithTimeoutException() throws IOException, AssertionErr
// Dropping master node to delay in cluster state call.
internalCluster().stopRandomNode(InternalTestCluster.nameFilter(masterNodes.get(0)));

CountDownLatch latch = new CountDownLatch(2);
new Thread(() -> {
try {
// Ensures the cancellation timeout expires.
Thread.sleep(2000);
// Starting master node to proceed in cluster state call.
internalCluster().startClusterManagerOnlyNode(
Settings.builder().put("node.name", masterNodes.get(0)).put(clusterManagerDataPathSettings).build()
);
latch.countDown();
} catch (InterruptedException e) {
throw new RuntimeException(e);
}
}).start();

CountDownLatch latch = new CountDownLatch(1);
final CatShardsRequest shardsRequest = new CatShardsRequest();
TimeValue timeoutInterval = timeValueMillis(1000);
shardsRequest.setCancelAfterTimeInterval(timeoutInterval);
shardsRequest.clusterManagerNodeTimeout(timeValueMillis(2500));
shardsRequest.setIndices(Strings.EMPTY_ARRAY);
client().execute(CatShardsAction.INSTANCE, shardsRequest, new ActionListener<CatShardsResponse>() {
client().execute(CatShardsAction.INSTANCE, shardsRequest, new ActionListener<>() {
@Override
public void onResponse(CatShardsResponse catShardsResponse) {
// onResponse should not be called.
Expand All @@ -132,7 +118,13 @@ public void onFailure(Exception e) {
latch.countDown();
}
});

latch.await();

// Restart cluster manager to restore cluster and allow test to cleanup and exit
internalCluster().startClusterManagerOnlyNode(
Settings.builder().put("node.name", masterNodes.get(0)).put(clusterManagerDataPathSettings).build()
);
}

public void testListShardsWithHiddenIndex() throws Exception {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -49,6 +49,7 @@
import org.opensearch.action.admin.indices.stats.IndicesStatsResponse;
import org.opensearch.cluster.metadata.RepositoryMetadata;
import org.opensearch.cluster.routing.allocation.decider.EnableAllocationDecider;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.unit.TimeValue;
import org.opensearch.core.common.unit.ByteSizeValue;
Expand Down Expand Up @@ -227,6 +228,7 @@ protected void setLowPriorityUploadRate(String repoName, String value) throws Ex
createRepository(repoName, rmd.type(), settings);
}

@SuppressForbidden(reason = "Waiting longer than timeout")
public void testCreateCloneIndexFailure() throws ExecutionException, InterruptedException {
asyncUploadMockFsRepo = false;
Version version = VersionUtils.randomIndexCompatibleVersion(random());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -136,13 +136,6 @@ public Map<String, Processor.Factory> getProcessors(Processor.Parameters paramet
public void execute(IngestDocument ingestDocument, BiConsumer<IngestDocument, Exception> handler) {
threadPool.generic().execute(() -> {
String id = (String) ingestDocument.getSourceAndMetadata().get("_id");
if (usually()) {
try {
Thread.sleep(10);
} catch (InterruptedException e) {
// ignore
}
}
ingestDocument.setFieldValue("foo", "bar-" + id);
handler.accept(ingestDocument, null);
});
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -44,6 +44,7 @@
import org.opensearch.cluster.routing.ShardRouting;
import org.opensearch.cluster.routing.allocation.decider.EnableAllocationDecider;
import org.opensearch.cluster.service.ClusterService;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.unit.TimeValue;
import org.opensearch.core.action.ActionListener;
Expand Down Expand Up @@ -149,6 +150,7 @@ private void setClusterInfoTimeout(String timeValue) {
);
}

@SuppressForbidden(reason = "Fixed sleeps are prone to flakiness but no documented failures here")
public void testClusterInfoServiceCollectsInformation() throws InterruptedException {
Settings.Builder settingsBuilder = Settings.builder()
.put(ResourceTrackerSettings.GLOBAL_JVM_USAGE_AC_WINDOW_DURATION_SETTING.getKey(), TimeValue.timeValueSeconds(1))
Expand Down Expand Up @@ -250,6 +252,7 @@ public void testClusterInfoServiceCollectsFileCacheInformation() {
}
}

@SuppressForbidden(reason = "Fixed sleeps are prone to flakiness but no documented failures here")
public void testClusterInfoServiceCollectsNodeResourceStatsInformation() throws InterruptedException {

// setting time window as ResourceUsageTracker needs atleast both of these to be ready to start collecting the
Expand Down Expand Up @@ -279,6 +282,7 @@ public void testClusterInfoServiceCollectsNodeResourceStatsInformation() throws
assertEquals(2, nodeResourceUsageStats.size());
}

@SuppressForbidden(reason = "Fixed sleeps are prone to flakiness but no documented failures here")
public void testClusterInfoServiceInformationClearOnError() throws InterruptedException {
internalCluster().startNodes(
2,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -40,6 +40,7 @@
import org.opensearch.cluster.NodeConnectionsService;
import org.opensearch.cluster.metadata.IndexMetadata;
import org.opensearch.cluster.service.ClusterService;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.settings.Settings;
import org.opensearch.index.MockEngineFactoryPlugin;
import org.opensearch.indices.recovery.RecoverySettings;
Expand Down Expand Up @@ -75,6 +76,7 @@
https://github.com/opensearch-project/OpenSearch/pull/15521 for context
*/
@ClusterScope(scope = Scope.TEST, numDataNodes = 0)
@SuppressForbidden(reason = "Pending fix: https://github.com/opensearch-project/OpenSearch/issues/18972")
public class NodeJoinLeftIT extends OpenSearchIntegTestCase {

private TestLogsAppender testLogsAppender;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -53,6 +53,7 @@
import org.opensearch.cluster.routing.allocation.AllocationService;
import org.opensearch.cluster.routing.allocation.ExistingShardsAllocator;
import org.opensearch.cluster.service.ClusterService;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.action.ActionFuture;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.unit.TimeValue;
Expand Down Expand Up @@ -219,6 +220,7 @@ public void testDeleteCreateInOneBulk() throws Exception {
assertHitCount(client().prepareSearch("test").get(), 0);
}

@SuppressForbidden(reason = "Best effort sleep, not critical to test logic")
public void testDelayedMappingPropagationOnPrimary() throws Exception {
// Here we want to test that things go well if there is a first request
// that adds mappings but before mappings are propagated to all nodes
Expand Down Expand Up @@ -309,6 +311,7 @@ public void testDelayedMappingPropagationOnPrimary() throws Exception {
assertEquals(1, docIndexResponse.get(10, TimeUnit.SECONDS).getShardInfo().getTotal());
}

@SuppressForbidden(reason = "Best effort sleep, not critical to test logic")
public void testDelayedMappingPropagationOnReplica() throws Exception {
// This is essentially the same thing as testDelayedMappingPropagationOnPrimary
// but for replicas
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -335,15 +335,14 @@ public void testGetWeightedRouting_ClusterManagerNotDiscovered() throws Exceptio
logger.info("--> network disruption is started");
networkDisruption.startDisrupting();

// wait for leader checker to fail
Thread.sleep(13000);

// get api to fetch local weighted routing for a node in zone a or b
ClusterGetWeightedRoutingResponse weightedRoutingResponse = internalCluster().client(
randomFrom(nodes_in_zone_a.get(0), nodes_in_zone_b.get(0))
).admin().cluster().prepareGetWeightedRouting().setAwarenessAttribute("zone").setRequestLocal(true).get();
assertEquals(weightedRouting, weightedRoutingResponse.weights());
assertFalse(weightedRoutingResponse.getDiscoveredClusterManager());
assertBusy(() -> {
ClusterGetWeightedRoutingResponse weightedRoutingResponse = internalCluster().client(
randomFrom(nodes_in_zone_a.get(0), nodes_in_zone_b.get(0))
).admin().cluster().prepareGetWeightedRouting().setAwarenessAttribute("zone").setRequestLocal(true).get();
assertEquals(weightedRouting, weightedRoutingResponse.weights());
assertFalse(weightedRoutingResponse.getDiscoveredClusterManager());
}, 13, TimeUnit.SECONDS);
logger.info("--> network disruption is stopped");
networkDisruption.stopDisrupting();

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -38,6 +38,7 @@
import org.opensearch.cluster.ClusterStateUpdateTask;
import org.opensearch.cluster.node.DiscoveryNode;
import org.opensearch.common.Nullable;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.unit.TimeValue;
import org.opensearch.test.OpenSearchIntegTestCase;
Expand All @@ -55,6 +56,8 @@
import static org.hamcrest.Matchers.equalTo;
import static org.hamcrest.Matchers.greaterThan;
import static org.hamcrest.Matchers.greaterThanOrEqualTo;
import static org.hamcrest.Matchers.hasSize;
import static org.awaitility.Awaitility.await;

@ClusterScope(scope = Scope.TEST, numDataNodes = 0)
public class ClusterServiceIT extends OpenSearchIntegTestCase {
Expand Down Expand Up @@ -350,6 +353,7 @@ public void testPendingUpdateTask() throws Exception {
final CountDownLatch invoked1 = new CountDownLatch(1);
clusterService.submitStateUpdateTask("1", new ClusterStateUpdateTask() {
@Override
@SuppressForbidden(reason = "Sleeping to guarantee a >0 time metric calculation")
public ClusterState execute(ClusterState currentState) {
try {
Thread.sleep(50);
Expand Down Expand Up @@ -458,10 +462,11 @@ public void onFailure(String source, Exception e) {
}
});
}
Thread.sleep(100);

pendingClusterTasks = clusterService.getClusterManagerService().pendingTasks();
assertThat(pendingClusterTasks.size(), greaterThanOrEqualTo(5));
pendingClusterTasks = await().until(
() -> clusterService.getClusterManagerService().pendingTasks(),
hasSize(greaterThanOrEqualTo(5))
);
controlSources = new HashSet<>(Arrays.asList("1", "2", "3", "4", "5"));
for (PendingClusterTask task : pendingClusterTasks) {
controlSources.remove(task.getSource().string());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -34,6 +34,7 @@

import org.opensearch.cluster.block.ClusterBlock;
import org.opensearch.cluster.block.ClusterBlockLevel;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.unit.TimeValue;
import org.opensearch.test.OpenSearchIntegTestCase;
Expand Down Expand Up @@ -74,6 +75,7 @@ public Client startNode(Settings.Builder settings) {
return internalCluster().client(name);
}

@SuppressForbidden(reason = "Sleeping longer than timeout")
public void testRecoverAfterNodes() throws Exception {
internalCluster().setBootstrapClusterManagerNodeIndex(0);
logger.info("--> start node (1)");
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,7 @@
import org.opensearch.cluster.node.DiscoveryNode;
import org.opensearch.cluster.node.DiscoveryNodes;
import org.opensearch.cluster.routing.ShardRouting;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.UUIDs;
import org.opensearch.common.action.ActionFuture;
import org.opensearch.common.collect.Tuple;
Expand All @@ -45,6 +46,7 @@
import static org.opensearch.test.hamcrest.OpenSearchAssertions.assertAcked;

@OpenSearchIntegTestCase.ClusterScope(scope = OpenSearchIntegTestCase.Scope.TEST, numDataNodes = 2, numClientNodes = 1)
@SuppressForbidden(reason = "Need to fix: https://github.com/opensearch-project/OpenSearch/issues/14331")
public class ShardIndexingPressureSettingsIT extends OpenSearchIntegTestCase {

public static final String INDEX_NAME = "test_index";
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -11,6 +11,7 @@
import com.carrotsearch.randomizedtesting.annotations.ThreadLeakFilters;

import org.opensearch.cluster.metadata.IndexMetadata;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.unit.TimeValue;
import org.opensearch.core.common.unit.ByteSizeUnit;
Expand Down Expand Up @@ -60,6 +61,7 @@ protected Settings nodeSettings(int nodeOrdinal) {
.build();
}

@SuppressForbidden(reason = "Sleeping longer than scheduled interval")
public void testAutoForceMergeFeatureFlagDisabled() throws InterruptedException, ExecutionException {

Settings clusterSettings = Settings.builder()
Expand Down Expand Up @@ -102,6 +104,7 @@ public void testAutoForceMergeFeatureFlagDisabled() throws InterruptedException,
assertAcked(client().admin().indices().prepareDelete(INDEX_NAME_1).get());
}

@SuppressForbidden(reason = "Sleeping longer than scheduled interval")
public void testAutoForceMergeTriggeringWithOneShardOfNonWarmCandidate() throws Exception {
Settings clusterSettings = Settings.builder()
.put(super.nodeSettings(0))
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -129,7 +129,6 @@ public void run() {
if (error.get() != null) {
throw error.get();
}
Thread.sleep(2000);
GetMappingsResponse mappings = client().admin().indices().prepareGetMappings("index").get();
for (int i = 0; i < indexThreads.length; ++i) {
assertMappingsHaveField(mappings, "index", "field" + i);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -37,6 +37,7 @@
import org.opensearch.action.support.replication.ReplicationResponse;
import org.opensearch.cluster.node.DiscoveryNode;
import org.opensearch.cluster.routing.ShardRouting;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.settings.Setting;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.unit.TimeValue;
Expand Down Expand Up @@ -216,6 +217,7 @@ public void testRetentionLeaseSyncedOnRemove() throws Exception {
}
}

@SuppressForbidden(reason = "Sleeping longer than lease duration")
public void testRetentionLeasesSyncOnExpiration() throws Exception {
final int numberOfReplicas = 2 - scaledRandomIntBetween(0, 2);
internalCluster().ensureAtLeastNumDataNodes(1 + numberOfReplicas);
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -54,6 +54,7 @@
import org.opensearch.cluster.metadata.IndexMetadata;
import org.opensearch.cluster.routing.allocation.command.MoveAllocationCommand;
import org.opensearch.cluster.routing.allocation.decider.EnableAllocationDecider;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.time.DateFormatter;
import org.opensearch.common.unit.TimeValue;
Expand Down Expand Up @@ -838,6 +839,7 @@ public void testTimedOutQuery() throws Exception {
protected Query doToQuery(QueryShardContext context) {
return new TermQuery(new Term("k", "hello")) {
@Override
@SuppressForbidden(reason = "Waiting 10x the timeout")
public Weight createWeight(IndexSearcher searcher, ScoreMode scoreMode, float boost) throws IOException {
// Create the weight before sleeping. Otherwise, TermStates.build() (in the call to super.createWeight()) will
// sometimes throw an exception on timeout, rather than timing out gracefully.
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -73,6 +73,7 @@
import org.opensearch.cluster.service.ClusterService;
import org.opensearch.common.Priority;
import org.opensearch.common.SetOnce;
import org.opensearch.common.SuppressForbidden;
import org.opensearch.common.concurrent.GatedCloseable;
import org.opensearch.common.settings.Settings;
import org.opensearch.common.unit.TimeValue;
Expand Down Expand Up @@ -1268,6 +1269,7 @@ public void testDisconnectsDuringRecovery() throws Exception {
redMockTransportService.addSendBehavior(blueMockTransportService, new StubbableTransport.SendRequestBehavior() {
private final AtomicInteger count = new AtomicInteger();

@SuppressForbidden(reason = "Simulating disconnect after sending request with a delay")
@Override
public void sendRequest(
Transport.Connection connection,
Expand Down
Loading
Loading