This is an automated email from the ASF dual-hosted git repository. jianliangqi pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/master by this push: new 7adab1138c [improvement](index) support CANCEL BUILD INDEX (#23760) 7adab1138c is described below commit 7adab1138cb8a50d678ac9536e53deb8a0e823c5 Author: Kang <kxiao.ti...@gmail.com> AuthorDate: Thu Sep 7 14:48:13 2023 +0800 [improvement](index) support CANCEL BUILD INDEX (#23760) Currently, BUILD INDEX can not be canceled. This pr add CANCEL BUILD INDEX as follows: `CANCEL BUILD INDEX index_name ON table_name [(jobid1,jobid2)]` --- docs/en/docs/data-table/index/inverted-index.md | 40 +++-- docs/zh-CN/docs/data-table/index/inverted-index.md | 34 ++-- fe/fe-core/src/main/cup/sql_parser.cup | 10 +- .../org/apache/doris/alter/IndexChangeJob.java | 37 ++++- .../java/org/apache/doris/alter/RollupJobV2.java | 2 +- .../apache/doris/alter/SchemaChangeHandler.java | 61 ++++++++ .../org/apache/doris/analysis/ShowAlterStmt.java | 2 +- .../java/org/apache/doris/catalog/DatabaseIf.java | 9 ++ .../main/java/org/apache/doris/catalog/Env.java | 3 +- .../inverted_index_p0/test_build_index.groovy | 172 +++++++++++++++++++++ 10 files changed, 331 insertions(+), 39 deletions(-) diff --git a/docs/en/docs/data-table/index/inverted-index.md b/docs/en/docs/data-table/index/inverted-index.md index c711b1fca4..1e17ca011b 100644 --- a/docs/en/docs/data-table/index/inverted-index.md +++ b/docs/en/docs/data-table/index/inverted-index.md @@ -119,22 +119,28 @@ ALTER TABLE table_name ADD INDEX idx_name(column_name) USING INVERTED [PROPERTIE **After version 2.0-beta (including 2.0-beta):** -The above 'create/add index' operation only generates inverted index for incremental data. The syntax of build index is added to add inverted index to stock data: +The above 'create/add index' operation only generates inverted index for incremental data. The syntax of BUILD INDEX is added to add inverted index to stock data: ```sql -- syntax 1, add inverted index to the stock data of the whole table by default BUILD INDEX index_name ON table_name; -- syntax 2, partition can be specified, and one or more can be specified BUILD INDEX index_name ON table_name PARTITIONS(partition_name1, partition_name2); ``` -(**The above 'create/add index' operation needs to be executed before executing the build index**) +(**The above 'create/add index' operation needs to be executed before executing the BUILD INDEX**) -To view the progress of the `build index`, you can use the following statement +To view the progress of the `BUILD INDEX`, you can run the following statement ```sql -show build index [FROM db_name]; --- Example 1: Viewing the progress of all build index tasks -show build index; --- Example 2: Viewing the progress of the build index task for a specified table -show build index where TableName = "table1"; +SHOW BUILD INDEX [FROM db_name]; +-- Example 1: Viewing the progress of all BUILD INDEX tasks +SHOW BUILD INDEX; +-- Example 2: Viewing the progress of the BUILD INDEX task for a specified table +SHOW BUILD INDEX where TableName = "table1"; +``` + +To cancel `BUILD INDEX`, you can run the following statement +```sql +CANCEL BUILD INDEX ON table_name; +CANCEL BUILD INDEX ON table_name (job_id1,jobid_2,...); ``` - drop an inverted index @@ -403,13 +409,13 @@ mysql> SELECT count() FROM hackernews_1m WHERE timestamp > '2007-08-23 04:17:00' mysql> CREATE INDEX idx_timestamp ON hackernews_1m(timestamp) USING INVERTED; Query OK, 0 rows affected (0.03 sec) ``` -**After 2.0-beta (including 2.0-beta), you need to execute `build index` to add inverted index to the stock data:** +**After 2.0-beta (including 2.0-beta), you need to execute `BUILD INDEX` to add inverted index to the stock data:** ```sql mysql> BUILD INDEX idx_timestamp ON hackernews_1m; Query OK, 0 rows affected (0.01 sec) ``` -- progress of building index can be view by SQL. It just costs 1s (compare FinishTime and CreateTime) to build index for timestamp column with 1 million rows. +- progress of building index can be view by SQL. It just costs 1s (compare FinishTime and CreateTime) to BUILD INDEX for timestamp column with 1 million rows. ```sql mysql> SHOW ALTER TABLE COLUMN; +-------+---------------+-------------------------+-------------------------+---------------+---------+---------------+---------------+---------------+----------+------+----------+---------+ @@ -420,10 +426,10 @@ mysql> SHOW ALTER TABLE COLUMN; 1 row in set (0.00 sec) ``` -**After 2.0-beta (including 2.0-beta), you can view the progress of stock data creating index by `show build index`:** +**After 2.0-beta (including 2.0-beta), you can view the progress of stock data creating index by `SHOW BUILD INDEX`:** ```sql -- If the table has no partitions, the PartitionName defaults to TableName -mysql> show build index; +mysql> SHOW BUILD INDEX; +-------+---------------+---------------+----------------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ | JobId | TableName | PartitionName | AlterInvertedIndexes | CreateTime | FinishTime | TransactionId | State | Msg | Progress | +-------+---------------+---------------+----------------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ @@ -458,7 +464,7 @@ mysql> SELECT count() FROM hackernews_1m WHERE parent = 11189; mysql> ALTER TABLE hackernews_1m ADD INDEX idx_parent(parent) USING INVERTED; Query OK, 0 rows affected (0.01 sec) --- After 2.0-beta (including 2.0-beta), you need to execute `build index` to add inverted index to the stock data: +-- After 2.0-beta (including 2.0-beta), you need to execute `BUILD INDEX` to add inverted index to the stock data: mysql> BUILD INDEX idx_parent ON hackernews_1m; Query OK, 0 rows affected (0.01 sec) @@ -470,7 +476,7 @@ mysql> SHOW ALTER TABLE COLUMN; | 10053 | hackernews_1m | 2023-02-10 19:49:32.893 | 2023-02-10 19:49:33.982 | hackernews_1m | 10054 | 10008 | 1:378856428 | 4 | FINISHED | | NULL | 2592000 | +-------+---------------+-------------------------+-------------------------+---------------+---------+---------------+---------------+---------------+----------+------+----------+---------+ -mysql> show build index; +mysql> SHOW BUILD INDEX; +-------+---------------+---------------+----------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ | JobId | TableName | PartitionName | AlterInvertedIndexes | CreateTime | FinishTime | TransactionId | State | Msg | Progress | +-------+---------------+---------------+----------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ @@ -501,11 +507,11 @@ mysql> SELECT count() FROM hackernews_1m WHERE author = 'faster'; mysql> ALTER TABLE hackernews_1m ADD INDEX idx_author(author) USING INVERTED; Query OK, 0 rows affected (0.01 sec) --- After 2.0-beta (including 2.0-beta), you need to execute `build index` to add inverted index to the stock data: +-- After 2.0-beta (including 2.0-beta), you need to execute `BUILD INDEX` to add inverted index to the stock data: mysql> BUILD INDEX idx_author ON hackernews_1m; Query OK, 0 rows affected (0.01 sec) --- costs 1.5s to build index for author column with 1 million rows. +-- costs 1.5s to BUILD INDEX for author column with 1 million rows. mysql> SHOW ALTER TABLE COLUMN; +-------+---------------+-------------------------+-------------------------+---------------+---------+---------------+---------------+---------------+----------+------+----------+---------+ | JobId | TableName | CreateTime | FinishTime | IndexName | IndexId | OriginIndexId | SchemaVersion | TransactionId | State | Msg | Progress | Timeout | @@ -515,7 +521,7 @@ mysql> SHOW ALTER TABLE COLUMN; | 10076 | hackernews_1m | 2023-02-10 19:54:20.046 | 2023-02-10 19:54:21.521 | hackernews_1m | 10077 | 10008 | 1:1335127701 | 5 | FINISHED | | NULL | 2592000 | +-------+---------------+-------------------------+-------------------------+---------------+---------+---------------+---------------+---------------+----------+------+----------+---------+ -mysql> show build index order by CreateTime desc limit 1; +mysql> SHOW BUILD INDEX order by CreateTime desc limit 1; +-------+---------------+---------------+----------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ | JobId | TableName | PartitionName | AlterInvertedIndexes | CreateTime | FinishTime | TransactionId | State | Msg | Progress | +-------+---------------+---------------+----------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ diff --git a/docs/zh-CN/docs/data-table/index/inverted-index.md b/docs/zh-CN/docs/data-table/index/inverted-index.md index 01ee5d1ea9..ce85973752 100644 --- a/docs/zh-CN/docs/data-table/index/inverted-index.md +++ b/docs/zh-CN/docs/data-table/index/inverted-index.md @@ -117,22 +117,28 @@ ALTER TABLE table_name ADD INDEX idx_name(column_name) USING INVERTED [PROPERTIE **2.0-beta版本(含2.0-beta)之后:** -上述`create/add index`操作只对增量数据生成倒排索引,增加了build index的语法用于对存量数据加倒排索引: +上述`create/add index`操作只对增量数据生成倒排索引,增加了BUILD INDEX的语法用于对存量数据加倒排索引: ```sql -- 语法1,默认给全表的存量数据加上倒排索引 BUILD INDEX index_name ON table_name; -- 语法2,可指定partition,可指定一个或多个 BUILD INDEX index_name ON table_name PARTITIONS(partition_name1, partition_name2); ``` -(**在执行build index之前需要已经执行了以上`create/add index`的操作**) +(**在执行BUILD INDEX之前需要已经执行了以上`create/add index`的操作**) -查看`build index`进展,可通过以下语句进行查看: +查看`BUILD INDEX`进展,可通过以下语句进行查看: ```sql -show build index [FROM db_name]; --- 示例1,查看所有的build index任务进展 -show build index; --- 示例2,查看指定table的build index任务进展 -show build index where TableName = "table1"; +SHOW BUILD INDEX [FROM db_name]; +-- 示例1,查看所有的BUILD INDEX任务进展 +SHOW BUILD INDEX; +-- 示例2,查看指定table的BUILD INDEX任务进展 +SHOW BUILD INDEX where TableName = "table1"; +``` + +取消 `BUILD INDEX`, 可通过以下语句进行 +```sql +CANCEL BUILD INDEX ON table_name; +CANCEL BUILD INDEX ON table_name (job_id1,jobid_2,...); ``` - 删除倒排索引 @@ -401,7 +407,7 @@ mysql> SELECT count() FROM hackernews_1m WHERE timestamp > '2007-08-23 04:17:00' mysql> CREATE INDEX idx_timestamp ON hackernews_1m(timestamp) USING INVERTED; Query OK, 0 rows affected (0.03 sec) ``` - **2.0-beta(含2.0-beta)后,需要再执行`build index`才能给存量数据加上倒排索引:** + **2.0-beta(含2.0-beta)后,需要再执行`BUILD INDEX`才能给存量数据加上倒排索引:** ```sql mysql> BUILD INDEX idx_timestamp ON hackernews_1m; Query OK, 0 rows affected (0.01 sec) @@ -421,7 +427,7 @@ mysql> SHOW ALTER TABLE COLUMN; **2.0-beta(含2.0-beta)后,可通过`show builde index`来查看存量数据创建索引进展:** ```sql -- 若table没有分区,PartitionName默认就是TableName -mysql> show build index; +mysql> SHOW BUILD INDEX; +-------+---------------+---------------+----------------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ | JobId | TableName | PartitionName | AlterInvertedIndexes | CreateTime | FinishTime | TransactionId | State | Msg | Progress | +-------+---------------+---------------+----------------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ @@ -456,7 +462,7 @@ mysql> SELECT count() FROM hackernews_1m WHERE parent = 11189; mysql> ALTER TABLE hackernews_1m ADD INDEX idx_parent(parent) USING INVERTED; Query OK, 0 rows affected (0.01 sec) --- 2.0-beta(含2.0-beta)后,需要再执行build index才能给存量数据加上倒排索引: +-- 2.0-beta(含2.0-beta)后,需要再执行BUILD INDEX才能给存量数据加上倒排索引: mysql> BUILD INDEX idx_parent ON hackernews_1m; Query OK, 0 rows affected (0.01 sec) @@ -468,7 +474,7 @@ mysql> SHOW ALTER TABLE COLUMN; | 10053 | hackernews_1m | 2023-02-10 19:49:32.893 | 2023-02-10 19:49:33.982 | hackernews_1m | 10054 | 10008 | 1:378856428 | 4 | FINISHED | | NULL | 2592000 | +-------+---------------+-------------------------+-------------------------+---------------+---------+---------------+---------------+---------------+----------+------+----------+---------+ -mysql> show build index; +mysql> SHOW BUILD INDEX; +-------+---------------+---------------+----------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ | JobId | TableName | PartitionName | AlterInvertedIndexes | CreateTime | FinishTime | TransactionId | State | Msg | Progress | +-------+---------------+---------------+----------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ @@ -499,7 +505,7 @@ mysql> SELECT count() FROM hackernews_1m WHERE author = 'faster'; mysql> ALTER TABLE hackernews_1m ADD INDEX idx_author(author) USING INVERTED; Query OK, 0 rows affected (0.01 sec) --- 2.0-beta(含2.0-beta)后,需要再执行build index才能给存量数据加上倒排索引: +-- 2.0-beta(含2.0-beta)后,需要再执行BUILD INDEX才能给存量数据加上倒排索引: mysql> BUILD INDEX idx_author ON hackernews_1m; Query OK, 0 rows affected (0.01 sec) @@ -513,7 +519,7 @@ mysql> SHOW ALTER TABLE COLUMN; | 10076 | hackernews_1m | 2023-02-10 19:54:20.046 | 2023-02-10 19:54:21.521 | hackernews_1m | 10077 | 10008 | 1:1335127701 | 5 | FINISHED | | NULL | 2592000 | +-------+---------------+-------------------------+-------------------------+---------------+---------+---------------+---------------+---------------+----------+------+----------+---------+ -mysql> show build index order by CreateTime desc limit 1; +mysql> SHOW BUILD INDEX order by CreateTime desc limit 1; +-------+---------------+---------------+----------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ | JobId | TableName | PartitionName | AlterInvertedIndexes | CreateTime | FinishTime | TransactionId | State | Msg | Progress | +-------+---------------+---------------+----------------------------------------------------+-------------------------+-------------------------+---------------+----------+------+----------+ diff --git a/fe/fe-core/src/main/cup/sql_parser.cup b/fe/fe-core/src/main/cup/sql_parser.cup index 2aa1a993d8..3cd7f9fd3f 100644 --- a/fe/fe-core/src/main/cup/sql_parser.cup +++ b/fe/fe-core/src/main/cup/sql_parser.cup @@ -855,7 +855,7 @@ nonterminal MultiPartitionDesc multi_partition_desc; nonterminal List<AccessPrivilegeWithCols> privilege_list; nonterminal List<String> string_list; -nonterminal List<Long> integer_list, cancel_rollup_job_id_list; +nonterminal List<Long> integer_list, job_id_list; nonterminal AccessPrivilegeWithCols privilege_type; nonterminal DataDescription data_desc, mysql_data_desc; @@ -4505,7 +4505,7 @@ cancel_stmt ::= :} ; -cancel_rollup_job_id_list ::= +job_id_list ::= {: RESULT = null; :} @@ -4524,10 +4524,14 @@ cancel_param ::= {: RESULT = new CancelExportStmt(db, parser.where); :} - | KW_ALTER KW_TABLE opt_alter_type:type KW_FROM table_name:table cancel_rollup_job_id_list:list + | KW_ALTER KW_TABLE opt_alter_type:type KW_FROM table_name:table job_id_list:list {: RESULT = new CancelAlterTableStmt(type, table, list); :} + | KW_BUILD KW_INDEX KW_ON table_name:table job_id_list:list + {: + RESULT = new CancelAlterTableStmt(ShowAlterStmt.AlterType.INDEX, table, list); + :} | KW_DECOMMISSION KW_BACKEND string_list:hostPorts {: RESULT = new CancelAlterSystemStmt(hostPorts); diff --git a/fe/fe-core/src/main/java/org/apache/doris/alter/IndexChangeJob.java b/fe/fe-core/src/main/java/org/apache/doris/alter/IndexChangeJob.java index 6ba3e68ffa..e6f4c4e0a0 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/alter/IndexChangeJob.java +++ b/fe/fe-core/src/main/java/org/apache/doris/alter/IndexChangeJob.java @@ -41,6 +41,7 @@ import org.apache.doris.task.AgentTaskExecutor; import org.apache.doris.task.AgentTaskQueue; import org.apache.doris.task.AlterInvertedIndexTask; import org.apache.doris.thrift.TColumn; +import org.apache.doris.thrift.TTaskType; import com.google.common.base.Joiner; import com.google.common.base.Preconditions; @@ -321,10 +322,31 @@ public class IndexChangeJob implements Writable { LOG.info("inverted index job finished: {}", jobId); } + /** + * cancelImpl() can be called any time any place. + * We need to clean any possible residual of this job. + */ protected boolean cancelImpl(String errMsg) { + if (jobState.isFinalState()) { + return false; + } + + cancelInternal(); + + jobState = JobState.CANCELLED; + this.errMsg = errMsg; + this.finishedTimeMs = System.currentTimeMillis(); + Env.getCurrentEnv().getEditLog().logIndexChangeJob(this); + LOG.info("cancel index job {}, err: {}", jobId, errMsg); return true; } + private void cancelInternal() { + // clear tasks if has + AgentTaskQueue.removeBatchTask(invertedIndexBatchTask, TTaskType.ALTER_INVERTED_INDEX); + // TODO maybe delete already build index files + } + public void replay(IndexChangeJob replayedJob) { try { IndexChangeJob replayedIndexChangeJob = (IndexChangeJob) replayedJob; @@ -336,8 +358,7 @@ public class IndexChangeJob implements Writable { replayRunningJob(replayedIndexChangeJob); break; case CANCELLED: - // TODO: - // replayCancelled(replayedIndexChangeJob); + replayCancelled(replayedIndexChangeJob); break; default: break; @@ -361,6 +382,18 @@ public class IndexChangeJob implements Writable { LOG.info("replay finished inverted index job: {} table id: {}", jobId, tableId); } + /** + * Replay job in CANCELLED state. + */ + private void replayCancelled(IndexChangeJob replayedJob) { + cancelInternal(); + + this.jobState = JobState.CANCELLED; + this.errMsg = replayedJob.errMsg; + this.finishedTimeMs = replayedJob.finishedTimeMs; + LOG.info("cancel index job {}, err: {}", jobId, errMsg); + } + public static IndexChangeJob read(DataInput in) throws IOException { if (Env.getCurrentEnvJournalVersion() < FeMetaVersion.VERSION_122) { IndexChangeJob job = new IndexChangeJob(); diff --git a/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java b/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java index f4f274e6d2..8d4af3648b 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java +++ b/fe/fe-core/src/main/java/org/apache/doris/alter/RollupJobV2.java @@ -639,8 +639,8 @@ public class RollupJobV2 extends AlterJobV2 implements GsonPostProcessable { jobState = JobState.CANCELLED; this.errMsg = errMsg; this.finishedTimeMs = System.currentTimeMillis(); - LOG.info("cancel {} job {}, err: {}", this.type, jobId, errMsg); Env.getCurrentEnv().getEditLog().logAlterJob(this); + LOG.info("cancel {} job {}, err: {}", this.type, jobId, errMsg); return true; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/alter/SchemaChangeHandler.java b/fe/fe-core/src/main/java/org/apache/doris/alter/SchemaChangeHandler.java index 8bd5fab817..3de1d43163 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/alter/SchemaChangeHandler.java +++ b/fe/fe-core/src/main/java/org/apache/doris/alter/SchemaChangeHandler.java @@ -32,6 +32,7 @@ import org.apache.doris.analysis.IndexDef.IndexType; import org.apache.doris.analysis.ModifyColumnClause; import org.apache.doris.analysis.ModifyTablePropertiesClause; import org.apache.doris.analysis.ReorderColumnsClause; +import org.apache.doris.analysis.ShowAlterStmt.AlterType; import org.apache.doris.catalog.AggregateType; import org.apache.doris.catalog.BinlogConfig; import org.apache.doris.catalog.Column; @@ -2336,7 +2337,14 @@ public class SchemaChangeHandler extends AlterHandler { @Override public void cancel(CancelStmt stmt) throws DdlException { CancelAlterTableStmt cancelAlterTableStmt = (CancelAlterTableStmt) stmt; + if (cancelAlterTableStmt.getAlterType() == AlterType.INDEX) { + cancelIndexJob(cancelAlterTableStmt); + } else { + cancelColumnJob(cancelAlterTableStmt); + } + } + private void cancelColumnJob(CancelAlterTableStmt cancelAlterTableStmt) throws DdlException { String dbName = cancelAlterTableStmt.getDbName(); String tableName = cancelAlterTableStmt.getTableName(); Preconditions.checkState(!Strings.isNullOrEmpty(dbName)); @@ -2375,6 +2383,59 @@ public class SchemaChangeHandler extends AlterHandler { } } + private void cancelIndexJob(CancelAlterTableStmt cancelAlterTableStmt) throws DdlException { + String dbName = cancelAlterTableStmt.getDbName(); + String tableName = cancelAlterTableStmt.getTableName(); + Preconditions.checkState(!Strings.isNullOrEmpty(dbName)); + Preconditions.checkState(!Strings.isNullOrEmpty(tableName)); + + Database db = Env.getCurrentInternalCatalog().getDbOrDdlException(dbName); + + List<IndexChangeJob> jobList = new ArrayList<>(); + + Table olapTable = db.getTableOrDdlException(tableName, Table.TableType.OLAP); + olapTable.writeLock(); + try { + // find from index change jobs first + if (cancelAlterTableStmt.getAlterJobIdList() != null + && cancelAlterTableStmt.getAlterJobIdList().size() > 0) { + for (Long jobId : cancelAlterTableStmt.getAlterJobIdList()) { + IndexChangeJob job = indexChangeJobs.get(jobId); + if (job == null) { + continue; + } + jobList.add(job); + LOG.debug("add build index job {} on table {} for specific id", jobId, tableName); + } + } else { + for (IndexChangeJob job : indexChangeJobs.values()) { + if (!job.isDone() && job.getTableId() == olapTable.getId()) { + jobList.add(job); + LOG.debug("add build index job {} on table {} for all", job.getJobId(), tableName); + } + } + } + } finally { + olapTable.writeUnlock(); + } + + // alter job v2's cancel must be called outside the table lock + if (jobList.size() > 0) { + for (IndexChangeJob job : jobList) { + long jobId = job.getJobId(); + LOG.debug("cancel build index job {} on table {}", jobId, tableName); + if (!job.cancel("user cancelled")) { + LOG.warn("cancel build index job {} on table {} failed", jobId, tableName); + throw new DdlException("Job can not be cancelled. State: " + job.getJobState()); + } else { + LOG.info("cancel build index job {} on table {} success", jobId, tableName); + } + } + } else { + throw new DdlException("No job to cancel for Table[" + tableName + "]"); + } + } + /** * Returns true if the index already exists, there is no need to create the job to add the index. * Otherwise return false, there is need to create a job to add the index. diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/ShowAlterStmt.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/ShowAlterStmt.java index 0b7017c49b..625d04d2dc 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/analysis/ShowAlterStmt.java +++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/ShowAlterStmt.java @@ -53,7 +53,7 @@ public class ShowAlterStmt extends ShowStmt { private static final Logger LOG = LogManager.getLogger(ShowAlterStmt.class); public static enum AlterType { - COLUMN, ROLLUP + COLUMN, ROLLUP, INDEX } private AlterType type; diff --git a/fe/fe-core/src/main/java/org/apache/doris/catalog/DatabaseIf.java b/fe/fe-core/src/main/java/org/apache/doris/catalog/DatabaseIf.java index 8c28337641..7dfceabbde 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/catalog/DatabaseIf.java +++ b/fe/fe-core/src/main/java/org/apache/doris/catalog/DatabaseIf.java @@ -202,6 +202,15 @@ public interface DatabaseIf<T extends TableIf> { return getTableOrException(tableName, t -> new DdlException(ErrorCode.ERR_BAD_TABLE_ERROR.formatErrorMsg(t))); } + default T getTableOrDdlException(String tableName, TableIf.TableType tableType) throws DdlException { + T table = getTableOrDdlException(tableName); + if (table.getType() != tableType) { + throw new DdlException( + "table type is not " + tableType + ", tableName=" + tableName + ", type=" + table.getType()); + } + return table; + } + default T getTableOrDdlException(long tableId) throws DdlException { return getTableOrException(tableId, t -> new DdlException(ErrorCode.ERR_BAD_TABLE_ERROR.formatErrorMsg(t))); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/catalog/Env.java b/fe/fe-core/src/main/java/org/apache/doris/catalog/Env.java index 4066b58775..873894e7df 100755 --- a/fe/fe-core/src/main/java/org/apache/doris/catalog/Env.java +++ b/fe/fe-core/src/main/java/org/apache/doris/catalog/Env.java @@ -4033,7 +4033,8 @@ public class Env { public void cancelAlter(CancelAlterTableStmt stmt) throws DdlException { if (stmt.getAlterType() == AlterType.ROLLUP) { this.getMaterializedViewHandler().cancel(stmt); - } else if (stmt.getAlterType() == AlterType.COLUMN) { + } else if (stmt.getAlterType() == AlterType.COLUMN + || stmt.getAlterType() == AlterType.INDEX) { this.getSchemaChangeHandler().cancel(stmt); } else { throw new DdlException("Cancel " + stmt.getAlterType() + " does not implement yet"); diff --git a/regression-test/suites/inverted_index_p0/test_build_index.groovy b/regression-test/suites/inverted_index_p0/test_build_index.groovy new file mode 100644 index 0000000000..13bf046592 --- /dev/null +++ b/regression-test/suites/inverted_index_p0/test_build_index.groovy @@ -0,0 +1,172 @@ +// 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. + + +suite("test_build_index", "inverted_index"){ + // prepare test table + def timeout = 60000 + def delta_time = 1000 + def alter_res = "null" + def useTime = 0 + + def wait_for_latest_op_on_table_finish = { table_name, OpTimeout -> + for(int t = delta_time; t <= OpTimeout; t += delta_time){ + alter_res = sql """SHOW ALTER TABLE COLUMN WHERE TableName = "${table_name}" ORDER BY CreateTime DESC LIMIT 1;""" + alter_res = alter_res.toString() + if(alter_res.contains("FINISHED")) { + sleep(3000) // wait change table state to normal + logger.info(table_name + " latest alter job finished, detail: " + alter_res) + break + } + useTime = t + sleep(delta_time) + } + assertTrue(useTime <= OpTimeout, "wait_for_latest_op_on_table_finish timeout") + } + + def wait_for_build_index_on_partition_finish = { table_name, OpTimeout -> + for(int t = delta_time; t <= OpTimeout; t += delta_time){ + alter_res = sql """SHOW BUILD INDEX WHERE TableName = "${table_name}";""" + def expected_finished_num = alter_res.size(); + def finished_num = 0; + for (int i = 0; i < expected_finished_num; i++) { + logger.info(table_name + " build index job state: " + alter_res[i][7] + i) + if (alter_res[i][7] == "FINISHED") { + ++finished_num; + } + } + if (finished_num == expected_finished_num) { + logger.info(table_name + " all build index jobs finished, detail: " + alter_res) + break + } + useTime = t + sleep(delta_time) + } + assertTrue(useTime <= OpTimeout, "wait_for_latest_build_index_on_partition_finish timeout") + } + + def wait_for_last_build_index_on_table_finish = { table_name, OpTimeout -> + for(int t = delta_time; t <= OpTimeout; t += delta_time){ + alter_res = sql """SHOW BUILD INDEX WHERE TableName = "${table_name}" ORDER BY JobId """ + + def last_job_state = alter_res[alter_res.size()-1][7]; + if (last_job_state == "FINISHED" || last_job_state == "CANCELLED") { + logger.info(table_name + " last index job finished, state: " + last_job_state + ", detail: " + alter_res) + return last_job_state; + } + useTime = t + sleep(delta_time) + } + assertTrue(useTime <= OpTimeout, "wait_for_last_build_index_on_table_finish timeout") + return "wait_timeout" + } + + def tableName = "hackernews_1m" + + sql "DROP TABLE IF EXISTS ${tableName}" + // create 1 replica table + sql """ + CREATE TABLE ${tableName} ( + `id` bigint(20) NULL, + `deleted` tinyint(4) NULL, + `type` text NULL, + `author` text NULL, + `timestamp` datetime NULL, + `comment` text NULL, + `dead` tinyint(4) NULL, + `parent` bigint(20) NULL, + `poll` bigint(20) NULL, + `children` array<bigint(20)> NULL, + `url` text NULL, + `score` int(11) NULL, + `title` text NULL, + `parts` array<int(11)> NULL, + `descendants` int(11) NULL + ) ENGINE=OLAP + DUPLICATE KEY(`id`) + COMMENT 'OLAP' + DISTRIBUTED BY HASH(`id`) BUCKETS 10 + PROPERTIES ( + "replication_allocation" = "tag.location.default: 1", + "is_being_synced" = "false", + "storage_format" = "V2", + "light_schema_change" = "true", + "disable_auto_compaction" = "false", + "enable_single_replica_compaction" = "false" + ); + """ + + // stream load data + streamLoad { + table "${tableName}" + + set 'compress_type', 'GZ' + + file """${getS3Url()}/regression/index/hacknernews_1m.csv.gz""" + + time 60000 // limit inflight 60s + + // stream load action will check result, include Success status, and NumberTotalRows == NumberLoadedRows + + // if declared a check callback, the default check condition will ignore. + // So you must check all condition + check { result, exception, startTime, endTime -> + if (exception != null) { + throw exception + } + log.info("Stream load result: ${result}".toString()) + def json = parseJson(result) + assertEquals("success", json.Status.toLowerCase()) + assertEquals(json.NumberTotalRows, json.NumberLoadedRows) + assertTrue(json.NumberLoadedRows > 0 && json.LoadBytes > 0) + } + } + + sql "sync" + + sql """ ALTER TABLE ${tableName} ADD INDEX idx_comment (`comment`) USING INVERTED PROPERTIES("parser" = "english") """ + + sql """ BUILD INDEX idx_comment ON ${tableName} """ + + sleep(1000) + + def result = sql """ SHOW BUILD INDEX WHERE TableName = "${tableName}" ORDER BY JobId """ + assertEquals(result[result.size()-1][1], tableName) + assertTrue(result[result.size()-1][3].contains("ADD INDEX")) + assertEquals(result[result.size()-1][7], "RUNNING") + + sql """ CANCEL BUILD INDEX ON ${tableName} (${result[result.size()-1][0]}) """ + result = sql """ SHOW BUILD INDEX WHERE TableName = "${tableName}" ORDER BY JobId """ + assertEquals(result[result.size()-1][1], tableName) + assertTrue(result[result.size()-1][3].contains("ADD INDEX")) + assertEquals(result[result.size()-1][7], "CANCELLED") + assertEquals(result[result.size()-1][8], "user cancelled") + + + sql """ BUILD INDEX idx_comment ON ${tableName}; """ + def state = wait_for_last_build_index_on_table_finish(tableName, timeout) + assertEquals(state, "FINISHED") + + def success = false; + try { + sql """ CANCEL BUILD INDEX ON ${tableName}; """ + success = true + } catch(Exception ex) { + logger.info(" CANCEL BUILD INDEX ON ${tableName} exception: " + ex) + } + assertFalse(success) +} --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org