Class SecondaryIndexManager

  • All Implemented Interfaces:
    IndexRegistry, INotificationConsumer

    public class SecondaryIndexManager
    extends java.lang.Object
    implements IndexRegistry, INotificationConsumer
    Handles the core maintenance functionality associated with indexes: adding/removing them to or from a table, (re)building during bootstrap or other streaming operations, flushing, reloading metadata and so on.

    The Index interface defines a number of methods which return Callable<?>. These are primarily the management tasks for an index implementation. Most of them are currently executed in a blocking fashion via submission to SIM's blockingExecutor. This provides the desired behaviour in pretty much all cases, as tasks like flushing an index needs to be executed synchronously to avoid potentially deadlocking on the FlushWriter or PostFlusher. Several of these Callable<?> returning methods on Index could then be defined with as void and called directly from SIM (rather than being run via the executor service). Separating the task defintion from execution gives us greater flexibility though, so that in future, for example, if the flush process allows it we leave open the possibility of executing more of these tasks asynchronously.

    The primary exception to the above is the Callable returned from Index#addIndexedColumn. This may involve a significant effort, building a new index over any existing data. We perform this task asynchronously; as it is called as part of a schema update, which we do not want to block for a long period. Building non-custom indexes is performed on the CompactionManager.

    This class also provides instances of processors which listen to updates to the base table and forward to registered Indexes the info required to keep those indexes up to date. There are two variants of these processors, each with a factory method provided by SIM: IndexTransaction: deals with updates generated on the regular write path. CleanupTransaction: used when partitions are modified during compaction or cleanup operations. Further details on their usage and lifecycles can be found in the interface definitions below.

    The bestIndexFor method is used at query time to identify the most selective index of those able to satisfy any search predicates defined by a ReadCommand's RowFilter. It returns a thin IndexAccessor object which enables the ReadCommand to access the appropriate functions of the Index at various stages in its lifecycle. e.g. the getEstimatedResultRows is required when StorageProxy calculates the initial concurrency factor for distributing requests to replicas, whereas a Searcher instance is needed when the ReadCommand is executed locally on a target replica.

    Finally, this class provides a clear and safe lifecycle to manage index builds, either full rebuilds via {@link this#rebuildIndexesBlocking(Set)} or builds of new sstables added via SSTableAddedNotifications, guaranteeing the following:
    • The initialization task and any subsequent successful (re)build mark the index as built.
    • If any (re)build operation fails, the index is not marked as built, and only another full rebuild can mark the index as built.
    • Full rebuilds cannot be run concurrently with other full or sstable (re)builds.
    • SSTable builds can always be run concurrently with any other builds.
    • Field Detail

      • baseCfs

        public final ColumnFamilyStore baseCfs
        The underlying column family containing the source data for these indexes
    • Constructor Detail

    • Method Detail

      • reload

        public void reload()
        Drops and adds new indexes associated with the underlying CF
      • buildIndex

        public Future<java.lang.Void> buildIndex​(Index index)
      • addIndex

        public Future<?> addIndex​(IndexMetadata indexDef,
                                  boolean isNewCF)
        Adds and builds a index
        Parameters:
        indexDef - the IndexMetadata describing the index
        isNewCF - true if the index is added as part of a new table/columnfamily (i.e. loading a CF at startup), false for all other cases (i.e. newly added index)
      • isIndexQueryable

        public boolean isIndexQueryable​(Index index)
        Checks if the specified index is queryable.
        Parameters:
        index - the index
        Returns:
        true if the specified index is queryable, false otherwise
      • isIndexWritable

        public boolean isIndexWritable​(Index index)
        Checks if the specified index is writable.
        Parameters:
        index - the index
        Returns:
        true if the specified index is writable, false otherwise
      • isIndexBuilding

        public boolean isIndexBuilding​(java.lang.String indexName)
        Checks if the specified index has any running build task.
        Parameters:
        indexName - the index name
        Returns:
        true if the index is building, false otherwise
      • removeIndex

        public void removeIndex​(java.lang.String indexName)
      • markAllIndexesRemoved

        public void markAllIndexesRemoved()
        Called when dropping a Table
      • rebuildIndexesBlocking

        public void rebuildIndexesBlocking​(java.util.Set<java.lang.String> indexNames)
        Does a blocking full rebuild/recovery of the specifed indexes from all the sstables in the base table. Note also that this method of (re)building/recovering indexes: a) takes a set of index *names* rather than Indexers b) marks existing indexes removed prior to rebuilding c) fails if such marking operation conflicts with any ongoing index builds, as full rebuilds cannot be run concurrently
        Parameters:
        indexNames - the list of indexes to be rebuilt
      • isIndexColumnFamilyStore

        public static boolean isIndexColumnFamilyStore​(ColumnFamilyStore cfs)
        Checks if the specified ColumnFamilyStore is a secondary index.
        Parameters:
        cfs - the ColumnFamilyStore to check.
        Returns:
        true if the specified ColumnFamilyStore is a secondary index, false otherwise.
      • isIndexColumnFamily

        public static boolean isIndexColumnFamily​(java.lang.String cfName)
        Checks if the specified ColumnFamilyStore is the one secondary index.
        Parameters:
        cfName - the name of the ColumnFamilyStore to check.
        Returns:
        true if the specified ColumnFamilyStore is a secondary index, false otherwise.
      • getParentCfsName

        public static java.lang.String getParentCfsName​(java.lang.String cfName)
        Returns the parent name of the specified ColumnFamilyStore.
        Parameters:
        cfName - the ColumnFamilyStore name
        Returns:
        the parent name of the specified ColumnFamilyStore
      • getIndexName

        public static java.lang.String getIndexName​(ColumnFamilyStore cfs)
        Returns the index name
        Parameters:
        cfs - the ColumnFamilyStore
        Returns:
        the index name
      • getIndexName

        public static java.lang.String getIndexName​(java.lang.String cfName)
        Returns the index name
        Parameters:
        cfName - the ColumnFamilyStore name
        Returns:
        the index name
      • validateSSTableAttachedIndexes

        public boolean validateSSTableAttachedIndexes​(java.util.Collection<SSTableReader> sstables,
                                                      boolean throwOnIncomplete,
                                                      boolean validateChecksum)
        Validates all index groups against the specified SSTables.
        Parameters:
        sstables - SSTables for which indexes in the group should be built
        throwOnIncomplete - whether to throw an error if any index in the group is incomplete
        validateChecksum - whether to validate checksum or not
        Returns:
        true if all indexes in all groups are complete and valid false if an index in any group is incomplete and throwOnIncomplete is false
        Throws:
        java.lang.IllegalStateException - if throwOnIncomplete is true and an index in any group is incomplete
        java.io.UncheckedIOException - if there is a problem validating any on-disk component in any group
      • markIndexesBuilding

        public void markIndexesBuilding​(java.util.Set<Index> indexes,
                                        boolean isFullRebuild,
                                        boolean isNewCF)
        Marks the specified indexes as (re)building if: 1) There's no in progress rebuild of any of the given indexes. 2) There's an in progress rebuild but the caller is not a full rebuild.

        Otherwise, this method invocation fails, as it is not possible to run full rebuilds while other concurrent rebuilds are in progress. Please note this is checked atomically against all given indexes; that is, no index will be marked if even a single one fails.

        Marking an index as "building" practically means: 1) The index is removed from the "failed" set if this is a full rebuild. 2) The index is removed from the system keyspace built indexes; this only happens if this method is not invoked for a new table initialization, as in such case there's no need to remove it (it is either already not present, or already present because already built).

        Thread safety is guaranteed by having all methods managing index builds synchronized: being synchronized on the SecondaryIndexManager instance, it means all invocations for all different indexes will go through the same lock, but this is fine as the work done while holding such lock is trivial.

        markIndexBuilt(Index, boolean) or markIndexFailed(Index, boolean) should be always called after the rebuilding has finished, so that the index build state can be correctly managed and the index rebuilt.

        Parameters:
        indexes - the index to be marked as building
        isFullRebuild - true if this method is invoked as a full index rebuild, false otherwise
        isNewCF - true if this method is invoked when initializing a new table/columnfamily (i.e. loading a CF at startup), false for all other cases (i.e. newly added index)
      • getIndexByName

        public Index getIndexByName​(java.lang.String indexName)
      • truncateAllIndexesBlocking

        public void truncateAllIndexesBlocking​(long truncatedAt)
        Truncate all indexes
      • dropAllIndexes

        public void dropAllIndexes​(boolean dropData)
        Remove all indexes
      • invalidateAllIndexesBlocking

        public void invalidateAllIndexesBlocking()
      • flushAllIndexesBlocking

        public void flushAllIndexesBlocking()
        Perform a blocking flush all indexes
      • flushIndexesBlocking

        public void flushIndexesBlocking​(java.util.Set<Index> indexes)
        Perform a blocking flush of selected indexes
      • executePreJoinTasksBlocking

        public void executePreJoinTasksBlocking​(boolean hadBootstrap)
        Performs a blocking execution of pre-join tasks of all indexes
      • flushAllNonCFSBackedIndexesBlocking

        public void flushAllNonCFSBackedIndexesBlocking​(Memtable baseCfsMemtable)
        Performs a blocking flush of all custom indexes
      • getBuiltIndexNames

        public java.util.List<java.lang.String> getBuiltIndexNames()
        Returns:
        all indexes which are marked as built and ready to use
      • getAllIndexColumnFamilyStores

        public java.util.Set<ColumnFamilyStore> getAllIndexColumnFamilyStores()
        Returns:
        all backing Tables used by registered indexes
      • hasIndexes

        public boolean hasIndexes()
        Returns:
        if there are ANY indexes registered for this table
      • indexPartition

        public void indexPartition​(DecoratedKey key,
                                   java.util.Set<Index> indexes,
                                   int pageSize)
      • indexPartition

        public void indexPartition​(DecoratedKey key,
                                   java.util.Set<Index> indexes,
                                   int pageSize,
                                   RegularAndStaticColumns columns)
        When building an index against existing data in sstables, add the given partition to the index
        Parameters:
        key - the key for the partition being indexed
        indexes - the indexes that must be updated
        pageSize - the number of Unfiltered objects to process in a single page
        columns - the columns indexed by at least one of the supplied indexes
      • calculateIndexingPageSize

        public int calculateIndexingPageSize()
        Return the page size used when indexing an entire partition
      • deletePartition

        public void deletePartition​(UnfilteredRowIterator partition,
                                    long nowInSec)
        Delete all data from all indexes for this partition. For when cleanup rips a partition out entirely.

        TODO : improve cleanup transaction to batch updates and perform them async

      • getBestIndexQueryPlanFor

        public Index.QueryPlan getBestIndexQueryPlanFor​(RowFilter rowFilter)
        Called at query time to choose which (if any) of the registered index implementations to use for a given query.

        This is a two step processes, firstly compiling the set of searchable indexes then choosing the one which reduces the search space the most.

        In the first phase, if the command's RowFilter contains any custom index expressions, the indexes that they specify are automatically included. Following that, the registered indexes are filtered to include only those which support the standard expressions in the RowFilter.

        The filtered set then sorted by selectivity, as reported by the Index implementations' getEstimatedResultRows method.

        Implementation specific validation of the target expression, either custom or standard, by the selected index should be performed in the searcherFor method to ensure that we pick the right index regardless of the validity of the expression.

        This method is only called once during the lifecycle of a ReadCommand and the result is cached for future use when obtaining a Searcher, getting the index's underlying CFS for ReadOrderGroup, or an estimate of the result size from an average index query.

        Parameters:
        rowFilter - RowFilter of the command to be executed
        Returns:
        the best available index query plan for the row filter, or null if none of the registered indexes can support the command.
      • getBestIndexFor

        public <T extends Index> java.util.Optional<T> getBestIndexFor​(RowFilter.Expression expression,
                                                                       java.lang.Class<T> indexType)
      • validate

        public void validate​(PartitionUpdate update,
                             ClientState state)
                      throws InvalidRequestException
        Called at write time to ensure that values present in the update are valid according to the rules of all registered indexes which will process it. The partition key as well as the clustering and cell values for each row in the update may be checked by index implementations
        Specified by:
        validate in interface IndexRegistry
        Parameters:
        update - PartitionUpdate containing the values to be validated by registered Index implementations
        state - state related to the client connection
        Throws:
        InvalidRequestException
      • needsFullRebuild

        public boolean needsFullRebuild​(java.lang.String index)
      • handles

        public boolean handles​(IndexTransaction.Type type)
        Parameters:
        type - index transaction type
        Returns:
        true if at least one of the indexes will be able to handle given index transaction type
      • shutdownAndWait

        public static void shutdownAndWait​(long timeout,
                                           java.util.concurrent.TimeUnit units)
                                    throws java.lang.InterruptedException,
                                           java.util.concurrent.TimeoutException
        Throws:
        java.lang.InterruptedException
        java.util.concurrent.TimeoutException
      • makeIndexNonQueryable

        public void makeIndexNonQueryable​(Index index,
                                          Index.Status status)
      • makeIndexQueryable

        public void makeIndexQueryable​(Index index,
                                       Index.Status status)