This is an automated email from the ASF dual-hosted git repository.

edimitrova pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/cassandra.git


The following commit(s) were added to refs/heads/trunk by this push:
     new 50978a0d07 CASSANDRA-20402: Add new reason 
RequestFailureReason.INDEX_BUILD_IN_PROGRESS and IndexBuildInProgress exception 
when queries fail during index build
50978a0d07 is described below

commit 50978a0d0738327290d06288c78967a61643506b
Author: Ekaterina Dimitrova <ekaterina.dimitr...@datastax.com>
AuthorDate: Thu Dec 5 16:28:08 2024 -0500

    CASSANDRA-20402: Add new reason RequestFailureReason.INDEX_BUILD_IN_PROGRESS
    and IndexBuildInProgress exception when queries fail during index build
    
    patch by Ekaterina Dimitrova; reviewed by Caleb Rackliffe for 
CASSANDRA-20402
---
 CHANGES.txt                                        |  1 +
 .../cassandra/exceptions/RequestFailureReason.java | 58 ++++++++--------
 .../index/IndexBuildInProgressException.java       | 36 ++++++++++
 .../apache/cassandra/index/IndexStatusManager.java | 12 +++-
 .../cassandra/index/SecondaryIndexManager.java     | 15 +++-
 src/java/org/apache/cassandra/net/InboundSink.java | 14 +++-
 .../test/sai/IndexAvailabilityTest.java            | 80 +++++++++++++++++++++-
 .../validation/entities/SecondaryIndexTest.java    | 13 ++--
 .../exceptions/RequestFailureReasonTest.java       | 43 +++++++++++-
 .../cassandra/index/IndexStatusManagerTest.java    |  2 +-
 .../index/sai/cql/AllowFilteringTest.java          | 32 +++++++++
 11 files changed, 267 insertions(+), 39 deletions(-)

diff --git a/CHANGES.txt b/CHANGES.txt
index 029a87ae29..c03ffcafa4 100644
--- a/CHANGES.txt
+++ b/CHANGES.txt
@@ -1,4 +1,5 @@
 5.1
+ * Throw new IndexBuildInProgressException when queries fail during index 
build, instead of IndexNotAvailableException (CASSANDRA-20402)
  * Fix Paxos repair interrupts running transactions (CASSANDRA-20469)
  * Various fixes in constraint framework (CASSANDRA-20481)
  * Add support in CAS for -= on numeric types, and fixed improper handling of 
empty bytes which lead to NPE (CASSANDRA-20477)
diff --git a/src/java/org/apache/cassandra/exceptions/RequestFailureReason.java 
b/src/java/org/apache/cassandra/exceptions/RequestFailureReason.java
index 1bc86ff061..9faff584f1 100644
--- a/src/java/org/apache/cassandra/exceptions/RequestFailureReason.java
+++ b/src/java/org/apache/cassandra/exceptions/RequestFailureReason.java
@@ -18,15 +18,18 @@
 package org.apache.cassandra.exceptions;
 
 import java.io.IOException;
+import java.util.HashMap;
+import java.util.Map;
 
 import org.apache.cassandra.db.filter.TombstoneOverwhelmingException;
+import org.apache.cassandra.index.IndexBuildInProgressException;
+import org.apache.cassandra.index.IndexNotAvailableException;
 import org.apache.cassandra.io.IVersionedSerializer;
 import org.apache.cassandra.io.util.DataInputPlus;
 import org.apache.cassandra.io.util.DataOutputPlus;
 import org.apache.cassandra.tcm.NotCMSException;
 import org.apache.cassandra.utils.vint.VIntCoding;
 
-import static java.lang.Math.max;
 import static org.apache.cassandra.net.MessagingService.VERSION_40;
 
 public enum RequestFailureReason
@@ -36,14 +39,16 @@ public enum RequestFailureReason
     TIMEOUT                  (2),
     INCOMPATIBLE_SCHEMA      (3),
     READ_SIZE                (4),
+    // below reason is only logged, but it does not have associated exception
     NODE_DOWN                (5),
     INDEX_NOT_AVAILABLE      (6),
+    // below reason does not have an associated exception
     READ_TOO_MANY_INDEXES    (7),
     NOT_CMS                  (8),
     INVALID_ROUTING          (9),
     COORDINATOR_BEHIND       (10),
-    ;
-
+    // The following codes have been ported from an external fork, where they 
were offset explicitly to avoid conflicts.
+    INDEX_BUILD_IN_PROGRESS  (503);
     public static final Serializer serializer = new Serializer();
 
     public final int code;
@@ -53,26 +58,32 @@ public enum RequestFailureReason
         this.code = code;
     }
 
-    private static final RequestFailureReason[] codeToReasonMap;
+    private static final Map<Integer, RequestFailureReason> codeToReasonMap = 
new HashMap<>();
+    private static final Map<Class<? extends Throwable>, RequestFailureReason> 
exceptionToReasonMap = new HashMap<>();
+    private static final int REASONS_WITHOUT_EXCEPTIONS = 3; // UNKNOWN, 
NODE_DOWN, and READ_TOO_MANY_INDEXES
 
     static
     {
         RequestFailureReason[] reasons = values();
 
-        int max = -1;
-        for (RequestFailureReason r : reasons)
-            max = max(r.code, max);
-
-        RequestFailureReason[] codeMap = new RequestFailureReason[max + 1];
-
         for (RequestFailureReason reason : reasons)
         {
-            if (codeMap[reason.code] != null)
+            if (codeToReasonMap.put(reason.code, reason) != null)
                 throw new RuntimeException("Two RequestFailureReason-s that 
map to the same code: " + reason.code);
-            codeMap[reason.code] = reason;
         }
 
-        codeToReasonMap = codeMap;
+        exceptionToReasonMap.put(TombstoneOverwhelmingException.class, 
READ_TOO_MANY_TOMBSTONES);
+        exceptionToReasonMap.put(WriteTimeoutException.class, TIMEOUT);
+        exceptionToReasonMap.put(IncompatibleSchemaException.class, 
INCOMPATIBLE_SCHEMA);
+        exceptionToReasonMap.put(ReadSizeAbortException.class, READ_SIZE);
+        exceptionToReasonMap.put(IndexNotAvailableException.class, 
INDEX_NOT_AVAILABLE);
+        exceptionToReasonMap.put(NotCMSException.class, NOT_CMS);
+        exceptionToReasonMap.put(InvalidRoutingException.class, 
INVALID_ROUTING);
+        exceptionToReasonMap.put(CoordinatorBehindException.class, 
COORDINATOR_BEHIND);
+        exceptionToReasonMap.put(IndexBuildInProgressException.class, 
INDEX_BUILD_IN_PROGRESS);
+
+        if (exceptionToReasonMap.size() != reasons.length - 
REASONS_WITHOUT_EXCEPTIONS)
+            throw new RuntimeException("A new RequestFailureReasons was 
probably added and you may need to update the exceptionToReasonMap");
     }
 
     public static RequestFailureReason fromCode(int code)
@@ -81,25 +92,18 @@ public enum RequestFailureReason
             throw new IllegalArgumentException("RequestFailureReason code must 
be non-negative (got " + code + ')');
 
         // be forgiving and return UNKNOWN if we aren't aware of the code - 
for forward compatibility
-        return code < codeToReasonMap.length ? codeToReasonMap[code] : UNKNOWN;
+        return codeToReasonMap.getOrDefault(code, UNKNOWN);
     }
 
     public static RequestFailureReason forException(Throwable t)
     {
-        if (t instanceof TombstoneOverwhelmingException)
-            return READ_TOO_MANY_TOMBSTONES;
-
-        if (t instanceof IncompatibleSchemaException)
-            return INCOMPATIBLE_SCHEMA;
-
-        if (t instanceof NotCMSException)
-            return NOT_CMS;
-
-        if (t instanceof InvalidRoutingException)
-            return INVALID_ROUTING;
+        RequestFailureReason r = exceptionToReasonMap.get(t.getClass());
+        if (r != null)
+            return r;
 
-        if (t instanceof CoordinatorBehindException)
-            return COORDINATOR_BEHIND;
+        for (Map.Entry<Class<? extends Throwable>, RequestFailureReason> entry 
: exceptionToReasonMap.entrySet())
+            if (entry.getKey().isInstance(t))
+                return entry.getValue();
 
         return UNKNOWN;
     }
diff --git 
a/src/java/org/apache/cassandra/index/IndexBuildInProgressException.java 
b/src/java/org/apache/cassandra/index/IndexBuildInProgressException.java
new file mode 100644
index 0000000000..1807ff36e0
--- /dev/null
+++ b/src/java/org/apache/cassandra/index/IndexBuildInProgressException.java
@@ -0,0 +1,36 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.cassandra.index;
+
+/**
+ * Thrown if a secondary index is not currently available because it is 
building.
+ */
+public final class IndexBuildInProgressException extends RuntimeException
+{
+    public static final String INDEX_BUILD_IN_PROGRESS_ERROR = "The secondary 
index '%s' is not yet available as it is building";
+
+    /**
+     * Creates a new <code>IndexIsBuildingException</code> for the specified 
index.
+     * @param index the index
+     */
+    public IndexBuildInProgressException(Index index)
+    {
+        super(String.format(INDEX_BUILD_IN_PROGRESS_ERROR, 
index.getIndexMetadata().name));
+    }
+}
diff --git a/src/java/org/apache/cassandra/index/IndexStatusManager.java 
b/src/java/org/apache/cassandra/index/IndexStatusManager.java
index cc98def63e..b11ecd1094 100644
--- a/src/java/org/apache/cassandra/index/IndexStatusManager.java
+++ b/src/java/org/apache/cassandra/index/IndexStatusManager.java
@@ -89,6 +89,7 @@ public class IndexStatusManager
     {
         // UNKNOWN states are transient/rare; only a few replicas should have 
this state at any time. See CASSANDRA-19400
         Set<Replica> queryableNonSucceeded = new HashSet<>(4);
+        Map<InetAddressAndPort, Index.Status> indexStatusMap = new HashMap<>();
 
         E queryableEndpoints = liveEndpoints.filter(replica -> {
 
@@ -97,7 +98,10 @@ public class IndexStatusManager
             {
                 Index.Status status = getIndexStatus(replica.endpoint(), 
keyspace.getName(), index.getIndexMetadata().name);
                 if (!index.isQueryable(status))
+                {
+                    indexStatusMap.put(replica.endpoint(), status);
                     return false;
+                }
 
                 if (status != Index.Status.BUILD_SUCCEEDED)
                     allBuilt = false;
@@ -125,7 +129,13 @@ public class IndexStatusManager
             {
                 Map<InetAddressAndPort, RequestFailureReason> failureReasons = 
new HashMap<>();
                 liveEndpoints.without(queryableEndpoints.endpoints())
-                             .forEach(replica -> 
failureReasons.put(replica.endpoint(), 
RequestFailureReason.INDEX_NOT_AVAILABLE));
+                             .forEach(replica -> {
+                                 Index.Status status = 
indexStatusMap.get(replica.endpoint());
+                                 if (status == 
Index.Status.FULL_REBUILD_STARTED)
+                                     failureReasons.put(replica.endpoint(), 
RequestFailureReason.INDEX_BUILD_IN_PROGRESS);
+                                 else
+                                     failureReasons.put(replica.endpoint(), 
RequestFailureReason.INDEX_NOT_AVAILABLE);
+                             });
 
                 throw new ReadFailureException(level, filtered, required, 
false, failureReasons);
             }
diff --git a/src/java/org/apache/cassandra/index/SecondaryIndexManager.java 
b/src/java/org/apache/cassandra/index/SecondaryIndexManager.java
index 791293fbb9..5f1c6e3d52 100644
--- a/src/java/org/apache/cassandra/index/SecondaryIndexManager.java
+++ b/src/java/org/apache/cassandra/index/SecondaryIndexManager.java
@@ -41,6 +41,7 @@ import com.google.common.collect.Lists;
 import com.google.common.collect.Maps;
 import com.google.common.collect.Sets;
 import com.google.common.util.concurrent.FutureCallback;
+import org.apache.cassandra.locator.InetAddressAndPort;
 import org.apache.commons.lang3.StringUtils;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
@@ -306,17 +307,29 @@ public class SecondaryIndexManager implements 
IndexRegistry, INotificationConsum
 
     /**
      * Throws an {@link IndexNotAvailableException} if any of the indexes in 
the specified {@link Index.QueryPlan} is
-     * not queryable, as it's defined by {@link #isIndexQueryable(Index)}.
+     * not queryable, as it's defined by {@link #isIndexQueryable(Index)}. If 
the reason for the index to be not available
+     * is that it's building, it will throw an {@link 
IndexBuildInProgressException}.
      *
      * @param queryPlan a query plan
      * @throws IndexNotAvailableException if the query plan has any index that 
is not queryable
      */
     public void checkQueryability(Index.QueryPlan queryPlan)
     {
+        InetAddressAndPort endpoint = FBUtilities.getBroadcastAddressAndPort();
+
         for (Index index : queryPlan.getIndexes())
         {
+            String indexName = index.getIndexMetadata().name;
+            Index.Status indexStatus = 
IndexStatusManager.instance.getIndexStatus(endpoint, keyspace.getName(), 
indexName);
+
             if (!isIndexQueryable(index))
+            {
+                // isQueryable is always true for non-SAI index 
implementations,  thus we need to check both not queryable and building
+                if (indexStatus == Index.Status.FULL_REBUILD_STARTED)
+                    throw new IndexBuildInProgressException(index);
+
                 throw new IndexNotAvailableException(index);
+            }
         }
     }
 
diff --git a/src/java/org/apache/cassandra/net/InboundSink.java 
b/src/java/org/apache/cassandra/net/InboundSink.java
index d077039635..2e8c8413dc 100644
--- a/src/java/org/apache/cassandra/net/InboundSink.java
+++ b/src/java/org/apache/cassandra/net/InboundSink.java
@@ -23,6 +23,7 @@ import java.util.concurrent.TimeUnit;
 import java.util.concurrent.atomic.AtomicReferenceFieldUpdater;
 import java.util.function.Predicate;
 
+import org.apache.cassandra.index.IndexBuildInProgressException;
 import org.slf4j.LoggerFactory;
 
 import net.openhft.chronicle.core.util.ThrowingConsumer;
@@ -126,13 +127,24 @@ public class InboundSink implements 
InboundMessageHandlers.MessageConsumer
             fail(message.header, t);
 
             if (t instanceof NotCMSException || t instanceof 
CoordinatorBehindException)
+            {
                 noSpamLogger.warn(t.getMessage());
-            else if (t instanceof TombstoneOverwhelmingException || t 
instanceof IndexNotAvailableException || t instanceof InvalidRoutingException)
+            }
+            else if (t instanceof TombstoneOverwhelmingException ||
+                     t instanceof IndexNotAvailableException ||
+                     t instanceof IndexBuildInProgressException ||
+                     t instanceof InvalidRoutingException)
+            {
                 noSpamLogger.error(t.getMessage());
+            }
             else if (t instanceof RuntimeException)
+            {
                 throw (RuntimeException) t;
+            }
             else
+            {
                 throw new RuntimeException(t);
+            }
         }
     }
 
diff --git 
a/test/distributed/org/apache/cassandra/distributed/test/sai/IndexAvailabilityTest.java
 
b/test/distributed/org/apache/cassandra/distributed/test/sai/IndexAvailabilityTest.java
index 53cca2614d..66a63fa11b 100644
--- 
a/test/distributed/org/apache/cassandra/distributed/test/sai/IndexAvailabilityTest.java
+++ 
b/test/distributed/org/apache/cassandra/distributed/test/sai/IndexAvailabilityTest.java
@@ -33,6 +33,7 @@ import net.bytebuddy.ByteBuddy;
 import net.bytebuddy.dynamic.loading.ClassLoadingStrategy;
 import net.bytebuddy.implementation.MethodDelegation;
 import org.apache.cassandra.distributed.Cluster;
+import org.apache.cassandra.distributed.api.ConsistencyLevel;
 import org.apache.cassandra.distributed.api.IInvokableInstance;
 import org.apache.cassandra.distributed.test.TestBaseImpl;
 import org.apache.cassandra.index.Index;
@@ -48,6 +49,7 @@ import static net.bytebuddy.matcher.ElementMatchers.named;
 import static org.apache.cassandra.distributed.api.Feature.GOSSIP;
 import static org.apache.cassandra.distributed.api.Feature.NETWORK;
 import static 
org.apache.cassandra.distributed.test.sai.SAIUtil.waitForIndexQueryable;
+import static org.assertj.core.api.Assertions.assertThatThrownBy;
 import static org.awaitility.Awaitility.await;
 import static org.junit.Assert.assertEquals;
 
@@ -57,7 +59,7 @@ public class IndexAvailabilityTest extends TestBaseImpl
     private static final String CREATE_TABLE = "CREATE TABLE %s.%s (pk text 
primary key, v1 int, v2 text) " +
                                                "WITH compaction = {'class' : 
'SizeTieredCompactionStrategy', 'enabled' : false }";
     private static final String CREATE_INDEX = "CREATE CUSTOM INDEX %s ON 
%s.%s(%s) USING 'StorageAttachedIndex'";
-    
+
     private static final Map<NodeIndex, Index.Status> 
expectedNodeIndexQueryability = new ConcurrentHashMap<>();
     private List<String> keyspaces;
     private List<String> indexesPerKs;
@@ -188,6 +190,82 @@ public class IndexAvailabilityTest extends TestBaseImpl
         });
     }
 
+    @Test
+    public void testIndexExceptionsTwoIndexesOn3NodeCluster() throws Exception
+    {
+        try (Cluster cluster = init(Cluster.build(3)
+                .withConfig(config -> config.with(GOSSIP)
+                                                          .with(NETWORK))
+                .start()))
+        {
+            String ks2 = "ks2";
+            String cf1 = "cf1";
+            String index1 = "cf1_idx1";
+            String index2 = "cf1_idx2";
+
+            // Create keyspace, table with correct column types
+            cluster.schemaChange(String.format(CREATE_KEYSPACE, ks2, 2));
+            cluster.schemaChange("CREATE TABLE " + ks2 + '.' + cf1 + " (pk int 
PRIMARY KEY, v1 int, v2 int)");
+            executeOnAllCoordinators(cluster,
+                              "SELECT pk FROM " + ks2 + '.' + cf1 + " WHERE 
v1=0 AND v2=0 ALLOW FILTERING");
+            executeOnAllCoordinators(cluster,
+                               "SELECT pk FROM " + ks2 + '.' + cf1 + " WHERE 
v2=0 ALLOW FILTERING");
+            executeOnAllCoordinators(cluster,
+                               "SELECT pk FROM " + ks2 + '.' + cf1 + " WHERE 
v1=0 ALLOW FILTERING");
+
+            cluster.schemaChange(String.format(CREATE_INDEX, index1, ks2, cf1, 
"v1"));
+            cluster.schemaChange(String.format(CREATE_INDEX, index2, ks2, cf1, 
"v2"));
+            cluster.forEach(node -> 
expectedNodeIndexQueryability.put(NodeIndex.create(ks2, index1, node), 
Index.Status.BUILD_SUCCEEDED));
+            for (IInvokableInstance node : cluster.get(2, 1, 3))
+                for (IInvokableInstance replica : cluster.get(1, 2, 3))
+                    waitForIndexingStatus(node, ks2, index1, replica, 
Index.Status.BUILD_SUCCEEDED);
+
+            // Mark only index2 as building on node3, leave index1 in 
BUILD_SUCCEEDED state
+            markIndexBuilding(cluster.get(3), ks2, cf1, index2);
+            cluster.forEach(node -> 
expectedNodeIndexQueryability.put(NodeIndex.create(ks2, index2, node), 
Index.Status.FULL_REBUILD_STARTED));
+            for (IInvokableInstance node : cluster.get(1, 2, 3))
+                waitForIndexingStatus(node, ks2, index2, cluster.get(3), 
Index.Status.FULL_REBUILD_STARTED);
+
+            assertThatThrownBy(() ->
+                    executeOnAllCoordinators(cluster,
+                                       "SELECT pk FROM " + ks2 + '.' + cf1 + " 
WHERE v1=0 AND v2=0"))
+                    .hasMessageContaining("Operation failed - received 1 
responses and 1 failures: INDEX_BUILD_IN_PROGRESS");
+
+            // Mark only index2 as failing on node2, leave index1 in 
BUILD_SUCCEEDED state
+            markIndexBuilding(cluster.get(2), ks2, cf1, index2);
+            cluster.forEach(node -> 
expectedNodeIndexQueryability.put(NodeIndex.create(ks2, index2, node), 
Index.Status.FULL_REBUILD_STARTED));
+            for (IInvokableInstance node : cluster.get(1, 2, 3))
+                waitForIndexingStatus(node, ks2, index2, cluster.get(2), 
Index.Status.FULL_REBUILD_STARTED);
+
+
+            assertThatThrownBy(() ->
+                    executeOnAllCoordinators(cluster,
+                                      "SELECT pk FROM " + ks2 + '.' + cf1 + " 
WHERE v1=0 AND v2=0"))
+                    .hasMessageContaining("Operation failed - received 1 
responses and 1 failures: INDEX_BUILD_IN_PROGRESS");
+
+            // Mark only index2 as failing on node1, leave index1 in 
BUILD_SUCCEEDED state
+            markIndexNonQueryable(cluster.get(1), ks2, cf1, index2);
+            cluster.forEach(node -> 
expectedNodeIndexQueryability.put(NodeIndex.create(ks2, index2, node), 
Index.Status.BUILD_FAILED));
+            for (IInvokableInstance node : cluster.get(1, 2, 3)) {
+                waitForIndexingStatus(node, ks2, index2, cluster.get(1), 
Index.Status.BUILD_FAILED);
+            }
+
+            assertThatThrownBy(() ->
+                    executeOnAllCoordinators(cluster,
+                                       "SELECT pk FROM " + ks2 + '.' + cf1 + " 
WHERE v1=0 AND v2=0"))
+                    .hasMessageMatching("^Operation failed - received 0 
responses and 2 failures: INDEX_NOT_AVAILABLE from .+, INDEX_BUILD_IN_PROGRESS 
from .+$");
+        }
+    }
+
+    private void executeOnAllCoordinators(Cluster cluster, String query)
+    {
+        // test different coordinator
+        for (int nodeId = 1; nodeId <= cluster.size(); nodeId++)
+        {
+            assertEquals(0, cluster.coordinator(nodeId).execute(query, 
ConsistencyLevel.LOCAL_QUORUM).length);
+        }
+    }
+
     @SuppressWarnings("DataFlowIssue")
     private void markIndexQueryable(IInvokableInstance node, String keyspace, 
String table, String indexName)
     {
diff --git 
a/test/unit/org/apache/cassandra/cql3/validation/entities/SecondaryIndexTest.java
 
b/test/unit/org/apache/cassandra/cql3/validation/entities/SecondaryIndexTest.java
index c1365e4cc3..6888ff3a93 100644
--- 
a/test/unit/org/apache/cassandra/cql3/validation/entities/SecondaryIndexTest.java
+++ 
b/test/unit/org/apache/cassandra/cql3/validation/entities/SecondaryIndexTest.java
@@ -25,12 +25,11 @@ import java.util.concurrent.Callable;
 import java.util.concurrent.CountDownLatch;
 
 import com.google.common.collect.ImmutableSet;
+
 import org.apache.commons.lang3.StringUtils;
 import org.junit.BeforeClass;
 import org.junit.Test;
 
-import org.apache.cassandra.index.internal.CassandraIndex;
-import org.apache.cassandra.index.sai.StorageAttachedIndex;
 import org.apache.cassandra.schema.ColumnMetadata;
 import org.apache.cassandra.schema.TableMetadata;
 import org.apache.cassandra.config.DatabaseDescriptor;
@@ -46,10 +45,12 @@ import org.apache.cassandra.db.rows.Cell;
 import org.apache.cassandra.db.rows.Row;
 import org.apache.cassandra.exceptions.ConfigurationException;
 import org.apache.cassandra.exceptions.SyntaxException;
-import org.apache.cassandra.index.IndexNotAvailableException;
+import org.apache.cassandra.index.IndexBuildInProgressException;
 import org.apache.cassandra.index.SecondaryIndexManager;
 import org.apache.cassandra.index.StubIndex;
+import org.apache.cassandra.index.internal.CassandraIndex;
 import org.apache.cassandra.index.internal.CustomCassandraIndex;
+import org.apache.cassandra.index.sai.StorageAttachedIndex;
 import org.apache.cassandra.index.sasi.SASIIndex;
 import org.apache.cassandra.schema.IndexMetadata;
 import org.apache.cassandra.service.ClientState;
@@ -1090,7 +1091,7 @@ public class SecondaryIndexTest extends CQLTester
             execute("SELECT value FROM %s WHERE value = 2");
             fail();
         }
-        catch (IndexNotAvailableException e)
+        catch (IndexBuildInProgressException e)
         {
             assertTrue(true);
         }
@@ -1124,7 +1125,7 @@ public class SecondaryIndexTest extends CQLTester
         indexName = createIndexAsync("CREATE CUSTOM INDEX ON %s (value) USING 
'" + ReadOnlyOnFailureIndex.class.getName() + "'");
         index = (ReadOnlyOnFailureIndex) 
getCurrentColumnFamilyStore().indexManager.getIndexByName(indexName);
         waitForIndexBuilds(indexName);
-        assertInvalidThrow(IndexNotAvailableException.class, "SELECT value 
FROM %s WHERE value = 1");
+        assertInvalidThrow(IndexBuildInProgressException.class, "SELECT value 
FROM %s WHERE value = 1");
         execute("INSERT INTO %s (pk, ck, value) VALUES (?, ?, ?)", 1, 1, 1);
         assertEquals(0, index.rowsInserted.size());
 
@@ -1164,7 +1165,7 @@ public class SecondaryIndexTest extends CQLTester
         waitForIndexBuilds(indexName);
         execute("INSERT INTO %s (pk, ck, value) VALUES (?, ?, ?)", 1, 1, 1);
         assertEquals(1, index.rowsInserted.size());
-        assertInvalidThrow(IndexNotAvailableException.class, "SELECT value 
FROM %s WHERE value = 1");
+        assertInvalidThrow(IndexBuildInProgressException.class, "SELECT value 
FROM %s WHERE value = 1");
 
         // Upon recovery, we can query data again
         index.reset();
diff --git 
a/test/unit/org/apache/cassandra/exceptions/RequestFailureReasonTest.java 
b/test/unit/org/apache/cassandra/exceptions/RequestFailureReasonTest.java
index b2fdcd365d..3b89fe9c64 100644
--- a/test/unit/org/apache/cassandra/exceptions/RequestFailureReasonTest.java
+++ b/test/unit/org/apache/cassandra/exceptions/RequestFailureReasonTest.java
@@ -20,8 +20,10 @@ package org.apache.cassandra.exceptions;
 
 import org.junit.Test;
 
+import static org.assertj.core.api.Assertions.assertThatThrownBy;
 import static org.junit.Assert.assertEquals;
 
+
 public class RequestFailureReasonTest
 {
     private static final RequestFailureReason[] REASONS = 
RequestFailureReason.values();
@@ -37,7 +39,8 @@ public class RequestFailureReasonTest
     { 7, "READ_TOO_MANY_INDEXES" },
     { 8, "NOT_CMS" },
     { 9, "INVALID_ROUTING" },
-    { 10, "COORDINATOR_BEHIND" }
+    { 10, "COORDINATOR_BEHIND" },
+    { 503, "INDEX_BUILD_IN_PROGRESS" }
     };
 
     @Test
@@ -54,4 +57,42 @@ public class RequestFailureReasonTest
         assertEquals("Number of RequestFailureReason enum constants has 
changed. Update the test.",
                      EXPECTED_VALUES.length, REASONS.length);
     }
+
+    @Test
+    public void testFromCode()
+    {
+        // Test valid codes
+        for (Object[] expected : EXPECTED_VALUES)
+        {
+            int code = (Integer) expected[0];
+            String name = (String) expected[1];
+            assertEquals(RequestFailureReason.valueOf(name), 
RequestFailureReason.fromCode(code));
+        }
+
+        // Test invalid codes
+        assertEquals(RequestFailureReason.UNKNOWN, 
RequestFailureReason.fromCode(200));
+        assertEquals(RequestFailureReason.UNKNOWN, 
RequestFailureReason.fromCode(999));
+        assertThatThrownBy(() -> 
RequestFailureReason.fromCode(-1)).isInstanceOf(IllegalArgumentException.class);
+    }
+
+    @Test
+    public void testExceptionSubclassMapping()
+    {
+        // Create a subclass of UnknownTableException
+        class CustomUnknownTableException extends IncompatibleSchemaException
+        {
+            public CustomUnknownTableException(String ks)
+            {
+                super(ks);
+            }
+        }
+        
+        // Verify the parent class still maps correctly
+        assertEquals(RequestFailureReason.INCOMPATIBLE_SCHEMA,
+                    RequestFailureReason.forException(new 
CustomUnknownTableException("ks")));
+        
+        // Test unmapped exception returns UNKNOWN
+        assertEquals(RequestFailureReason.UNKNOWN, 
+                    RequestFailureReason.forException(new 
RuntimeException("test")));
+    }
 }
diff --git a/test/unit/org/apache/cassandra/index/IndexStatusManagerTest.java 
b/test/unit/org/apache/cassandra/index/IndexStatusManagerTest.java
index 947b7a57bc..39401ac1bc 100644
--- a/test/unit/org/apache/cassandra/index/IndexStatusManagerTest.java
+++ b/test/unit/org/apache/cassandra/index/IndexStatusManagerTest.java
@@ -359,7 +359,7 @@ public class IndexStatusManagerTest
                 .hasMessageStartingWith("Operation failed")
                 .hasMessageContaining("INDEX_NOT_AVAILABLE from 
/127.0.0.253:7000")
                 .hasMessageContaining("INDEX_NOT_AVAILABLE from 
/127.0.0.254:7000")
-                .hasMessageContaining("INDEX_NOT_AVAILABLE from 
/127.0.0.255:7000");
+                .hasMessageContaining("INDEX_BUILD_IN_PROGRESS from 
/127.0.0.255:7000");
     }
 
     void runTest(Testcase testcase)
diff --git 
a/test/unit/org/apache/cassandra/index/sai/cql/AllowFilteringTest.java 
b/test/unit/org/apache/cassandra/index/sai/cql/AllowFilteringTest.java
index 7a9198a700..3ae0905185 100644
--- a/test/unit/org/apache/cassandra/index/sai/cql/AllowFilteringTest.java
+++ b/test/unit/org/apache/cassandra/index/sai/cql/AllowFilteringTest.java
@@ -21,10 +21,14 @@ package org.apache.cassandra.index.sai.cql;
 import org.junit.Test;
 
 import org.apache.cassandra.cql3.restrictions.StatementRestrictions;
+import org.apache.cassandra.index.IndexBuildInProgressException;
 import org.apache.cassandra.index.sai.SAITester;
 import org.apache.cassandra.index.sai.StorageAttachedIndex;
+import org.apache.cassandra.inject.Injections;
+import org.apache.cassandra.inject.InvokePointBuilder;
 
 import static java.lang.String.format;
+import static org.assertj.core.api.Assertions.assertThatThrownBy;
 import static org.junit.Assert.assertNotNull;
 
 /**
@@ -391,4 +395,32 @@ public class AllowFilteringTest extends SAITester
         assertNotNull(execute(query + " ALLOW FILTERING"));
     }
 
+    private static final Injections.Barrier blockIndexBuild = 
Injections.newBarrier("block_index_build", 2, false)
+                                                                        
.add(InvokePointBuilder.newInvokePoint()
+                                                                        
.onClass(StorageAttachedIndex.class)
+                                                                        
.onMethod("startInitialBuild"))
+                                                                        
.build();
+
+    @Test
+    public void testAllowFilteringDuringIndexBuild() throws Throwable
+    {
+        createTable("CREATE TABLE %s (k int PRIMARY KEY, v int)");
+        Injections.inject(blockIndexBuild);
+        String idx = createIndexAsync(String.format("CREATE CUSTOM INDEX ON 
%%s(v) USING '%s'", StorageAttachedIndex.class.getName()));
+
+        String expectedErrorMessage = 
String.format(IndexBuildInProgressException.INDEX_BUILD_IN_PROGRESS_ERROR, idx);
+        assertThatThrownBy(() -> execute("SELECT * FROM %s WHERE v=0"))
+                .hasMessage(expectedErrorMessage)
+                .isInstanceOf(IndexBuildInProgressException.class);
+
+        assertThatThrownBy(() -> execute("SELECT * FROM %s WHERE v=0 ALLOW 
FILTERING"))
+                .hasMessage(expectedErrorMessage)
+                .isInstanceOf(IndexBuildInProgressException.class);
+
+        blockIndexBuild.countDown();
+        blockIndexBuild.disable();
+        waitForIndexQueryable(idx);
+        execute("SELECT * FROM %s WHERE v=0");
+        execute("SELECT * FROM %s WHERE v=0 ALLOW FILTERING");
+    }
 }


---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscr...@cassandra.apache.org
For additional commands, e-mail: commits-h...@cassandra.apache.org

Reply via email to