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

lijibing 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 e8c44a7fee1 [feature](neredis)Support show analyze in Nereids. (#47280)
e8c44a7fee1 is described below

commit e8c44a7fee11da5146942b3a2f76f459cfabdfdc
Author: James <lijib...@selectdb.com>
AuthorDate: Mon Feb 10 16:55:51 2025 +0800

    [feature](neredis)Support show analyze in Nereids. (#47280)
    
    ### What problem does this PR solve?
    
    Support show analyze in Nereids.
---
 .../antlr4/org/apache/doris/nereids/DorisParser.g4 |   8 +-
 .../doris/nereids/parser/LogicalPlanBuilder.java   |  12 ++
 .../apache/doris/nereids/trees/plans/PlanType.java |   1 +
 .../trees/plans/commands/ShowAnalyzeCommand.java   | 206 +++++++++++++++++++++
 .../trees/plans/visitor/CommandVisitor.java        |   5 +
 .../apache/doris/statistics/AnalysisManager.java   |  35 ++--
 .../suites/statistics/analyze_stats.groovy         |  24 ++-
 7 files changed, 277 insertions(+), 14 deletions(-)

diff --git a/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 
b/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4
index 25c7d649777..71bd2999884 100644
--- a/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4
+++ b/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4
@@ -64,6 +64,7 @@ statementBase
     | supportedAdminStatement           #supportedAdminStatementAlias
     | supportedUseStatement             #supportedUseStatementAlias
     | supportedOtherStatement           #supportedOtherStatementAlias
+    | supportedStatsStatement           #supportedStatsStatementAlias
     | unsupportedStatement              #unsupported
     ;
 
@@ -710,6 +711,11 @@ unsupportedDropStatement
     | DROP STAGE (IF EXISTS)? name=identifier                                  
 #dropStage
     ;
 
+supportedStatsStatement
+    : SHOW AUTO? ANALYZE (jobId=INTEGER_VALUE | tableName=multipartIdentifier)?
+        (WHERE (stateKey=identifier) EQ (stateValue=STRING_LITERAL))?          
 #showAnalyze
+    ;
+
 unsupportedStatsStatement
     : ANALYZE TABLE name=multipartIdentifier partitionSpec?
         columns=identifierList? (WITH analyzeProperties)* propertyClause?      
 #analyzeTable
@@ -734,8 +740,6 @@ unsupportedStatsStatement
         columnList=identifierList? partitionSpec?                              
 #showColumnStats
     | SHOW COLUMN HISTOGRAM tableName=multipartIdentifier
         columnList=identifierList                                              
 #showColumnHistogramStats
-    | SHOW AUTO? ANALYZE tableName=multipartIdentifier? wildWhere?             
 #showAnalyze
-    | SHOW ANALYZE jobId=INTEGER_VALUE wildWhere?                              
 #showAnalyzeFromJobId
     | SHOW AUTO JOBS tableName=multipartIdentifier? wildWhere?                 
 #showAutoAnalyzeJobs
     | SHOW ANALYZE TASK STATUS jobId=INTEGER_VALUE                             
 #showAnalyzeTask
     ;
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java
index 1a30545f925..bec8bc4d617 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java
@@ -275,6 +275,7 @@ import 
org.apache.doris.nereids.DorisParser.SetUserPropertiesContext;
 import org.apache.doris.nereids.DorisParser.SetUserVariableContext;
 import org.apache.doris.nereids.DorisParser.SetVariableWithTypeContext;
 import org.apache.doris.nereids.DorisParser.ShowAllPropertiesContext;
+import org.apache.doris.nereids.DorisParser.ShowAnalyzeContext;
 import org.apache.doris.nereids.DorisParser.ShowAuthorsContext;
 import org.apache.doris.nereids.DorisParser.ShowBackendsContext;
 import org.apache.doris.nereids.DorisParser.ShowBrokerContext;
@@ -564,6 +565,7 @@ import 
org.apache.doris.nereids.trees.plans.commands.SetDefaultStorageVaultComma
 import org.apache.doris.nereids.trees.plans.commands.SetOptionsCommand;
 import org.apache.doris.nereids.trees.plans.commands.SetTransactionCommand;
 import org.apache.doris.nereids.trees.plans.commands.SetUserPropertiesCommand;
+import org.apache.doris.nereids.trees.plans.commands.ShowAnalyzeCommand;
 import org.apache.doris.nereids.trees.plans.commands.ShowAuthorsCommand;
 import org.apache.doris.nereids.trees.plans.commands.ShowBackendsCommand;
 import org.apache.doris.nereids.trees.plans.commands.ShowBrokerCommand;
@@ -5522,5 +5524,15 @@ public class LogicalPlanBuilder extends 
DorisParserBaseVisitor<Object> {
 
         return new AlterRepositoryCommand(ctx.name.getText(), properties);
     }
+
+    @Override
+    public LogicalPlan visitShowAnalyze(ShowAnalyzeContext ctx) {
+        boolean isAuto = ctx.AUTO() != null;
+        List<String> tableName = ctx.tableName == null ? null : 
visitMultipartIdentifier(ctx.tableName);
+        long jobId = ctx.jobId == null ? 0 : 
Long.parseLong(ctx.jobId.getText());
+        String stateKey = ctx.stateKey == null ? null : 
stripQuotes(ctx.stateKey.getText());
+        String stateValue = ctx.stateValue == null ? null : 
stripQuotes(ctx.stateValue.getText());
+        return new ShowAnalyzeCommand(tableName, jobId, stateKey, stateValue, 
isAuto);
+    }
 }
 
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/PlanType.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/PlanType.java
index 91820211d1e..6ee7ad6aa65 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/PlanType.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/PlanType.java
@@ -214,6 +214,7 @@ public enum PlanType {
     ALTER_CATALOG_COMMENT_COMMAND,
     ALTER_SQL_BLOCK_RULE_COMMAND,
     ALTER_REPOSITORY_COMMAND,
+    SHOW_ANALYZE_COMMAND,
     SHOW_BACKENDS_COMMAND,
     SHOW_BLOCK_RULE_COMMAND,
     SHOW_BROKER_COMMAND,
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/ShowAnalyzeCommand.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/ShowAnalyzeCommand.java
new file mode 100644
index 00000000000..8fe305353e1
--- /dev/null
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/ShowAnalyzeCommand.java
@@ -0,0 +1,206 @@
+// 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.doris.nereids.trees.plans.commands;
+
+import org.apache.doris.analysis.RedirectStatus;
+import org.apache.doris.catalog.Column;
+import org.apache.doris.catalog.DatabaseIf;
+import org.apache.doris.catalog.Env;
+import org.apache.doris.catalog.ScalarType;
+import org.apache.doris.catalog.TableIf;
+import org.apache.doris.common.AnalysisException;
+import org.apache.doris.common.ErrorCode;
+import org.apache.doris.common.ErrorReport;
+import org.apache.doris.common.util.TimeUtils;
+import org.apache.doris.datasource.CatalogIf;
+import org.apache.doris.mysql.privilege.PrivPredicate;
+import org.apache.doris.nereids.trees.plans.PlanType;
+import org.apache.doris.nereids.trees.plans.commands.info.TableNameInfo;
+import org.apache.doris.nereids.trees.plans.visitor.PlanVisitor;
+import org.apache.doris.qe.ConnectContext;
+import org.apache.doris.qe.ShowResultSet;
+import org.apache.doris.qe.ShowResultSetMetaData;
+import org.apache.doris.qe.StmtExecutor;
+import org.apache.doris.statistics.AnalysisInfo;
+import org.apache.doris.statistics.util.StatisticsUtil;
+
+import com.google.common.collect.Lists;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.time.Instant;
+import java.time.LocalDateTime;
+import java.time.ZoneId;
+import java.time.format.DateTimeFormatter;
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Optional;
+
+/**
+ * show analyze command.
+ */
+public class ShowAnalyzeCommand extends ShowCommand {
+    private static final Logger LOG = 
LogManager.getLogger(ShowAnalyzeCommand.class);
+
+    private static final String STATE_NAME = "state";
+    private static final ShowResultSetMetaData META_DATA =
+            ShowResultSetMetaData.builder()
+                    .addColumn(new Column("job_id", 
ScalarType.createVarchar(128)))
+                    .addColumn(new Column("catalog_name", 
ScalarType.createStringType()))
+                    .addColumn(new Column("db_name", 
ScalarType.createStringType()))
+                    .addColumn(new Column("tbl_name", 
ScalarType.createStringType()))
+                    .addColumn(new Column("col_name", 
ScalarType.createStringType()))
+                    .addColumn(new Column("job_type", 
ScalarType.createVarchar(64)))
+                    .addColumn(new Column("analysis_type", 
ScalarType.createVarchar(64)))
+                    .addColumn(new Column("message", 
ScalarType.createStringType()))
+                    .addColumn(new Column("last_exec_time_in_ms", 
ScalarType.createStringType()))
+                    .addColumn(new Column("state", 
ScalarType.createVarchar(32)))
+                    .addColumn(new Column("progress", 
ScalarType.createStringType()))
+                    .addColumn(new Column("schedule_type", 
ScalarType.createStringType()))
+                    .addColumn(new Column("start_time", 
ScalarType.createStringType()))
+                    .addColumn(new Column("end_time", 
ScalarType.createStringType()))
+                    .addColumn(new Column("priority", 
ScalarType.createStringType()))
+                    .addColumn(new Column("enable_partition", 
ScalarType.createVarchar(32)))
+                    .build();
+
+    private final TableNameInfo tableNameInfo;
+    private final long jobId;
+    private final String stateKey;
+    private final String stateValue;
+    private final boolean isAuto;
+    private String ctl;
+    private String db;
+    private String table;
+
+    /**
+     * Constructor.
+     * @param tableName catalog.db.table
+     * @param jobId analyze job id.
+     * @param stateKey Filter column name, Only support "state" for now."
+     * @param stateValue Filter column value. Only support 
STATE="PENDING|RUNNING|FINISHED|FAILED"
+     * @param isAuto show auto analyze or manual analyze.
+     */
+    public ShowAnalyzeCommand(List<String> tableName, long jobId, String 
stateKey, String stateValue, boolean isAuto) {
+        super(PlanType.SHOW_ANALYZE_COMMAND);
+        this.tableNameInfo = tableName == null ? null : new 
TableNameInfo(tableName);
+        this.jobId = jobId;
+        this.stateKey = stateKey;
+        this.stateValue = stateValue;
+        this.isAuto = isAuto;
+        this.ctl = null;
+        this.db = null;
+        this.table = null;
+    }
+
+    private void validate(ConnectContext ctx) throws AnalysisException {
+        checkShowAnalyzePriv(ctx);
+        if (tableNameInfo != null) {
+            tableNameInfo.analyze(ctx);
+            ctl = tableNameInfo.getCtl();
+            db = tableNameInfo.getDb();
+            table = tableNameInfo.getTbl();
+        }
+        if (stateKey == null && stateValue != null || stateKey != null && 
stateValue == null) {
+            throw new AnalysisException("Invalid where clause, should be STATE 
= \"PENDING|RUNNING|FINISHED|FAILED\"");
+        }
+        if (stateKey != null) {
+            if (!stateKey.equalsIgnoreCase(STATE_NAME)
+                    || !stateValue.equalsIgnoreCase("PENDING")
+                    && !stateValue.equalsIgnoreCase("RUNNING")
+                    && !stateValue.equalsIgnoreCase("FINISHED")
+                    && !stateValue.equalsIgnoreCase("FAILED")) {
+                throw new AnalysisException("Where clause should be STATE = 
\"PENDING|RUNNING|FINISHED|FAILED\"");
+            }
+        }
+    }
+
+    private void checkShowAnalyzePriv(ConnectContext ctx) throws 
AnalysisException {
+        if (!Env.getCurrentEnv().getAccessManager().checkGlobalPriv(ctx, 
PrivPredicate.SHOW)) {
+            ErrorReport.reportAnalysisException(
+                    ErrorCode.ERR_ACCESS_DENIED_ERROR,
+                    "SHOW ANALYZE",
+                    ConnectContext.get().getQualifiedUser(),
+                    ConnectContext.get().getRemoteIP());
+        }
+    }
+
+    @Override
+    public <R, C> R accept(PlanVisitor<R, C> visitor, C context) {
+        return visitor.visitShowAnalyzeCommand(this, context);
+    }
+
+    @Override
+    public ShowResultSet doRun(ConnectContext ctx, StmtExecutor executor) 
throws Exception {
+        validate(ctx);
+        return handleShowAnalyze();
+    }
+
+    private ShowResultSet handleShowAnalyze() {
+        List<AnalysisInfo> results = Env.getCurrentEnv().getAnalysisManager()
+                .findAnalysisJobs(stateValue, ctl, db, table, jobId, isAuto);
+        List<List<String>> resultRows = Lists.newArrayList();
+        DateTimeFormatter formatter = DateTimeFormatter.ofPattern("yyyy-MM-dd 
HH:mm:ss");
+        for (AnalysisInfo analysisInfo : results) {
+            try {
+                List<String> row = new ArrayList<>();
+                row.add(String.valueOf(analysisInfo.jobId));
+                CatalogIf<? extends DatabaseIf<? extends TableIf>> c
+                        = StatisticsUtil.findCatalog(analysisInfo.catalogId);
+                row.add(c.getName());
+                Optional<? extends DatabaseIf<? extends TableIf>> databaseIf = 
c.getDb(analysisInfo.dbId);
+                row.add(databaseIf.isPresent() ? 
databaseIf.get().getFullName() : "DB may get deleted");
+                if (databaseIf.isPresent()) {
+                    Optional<? extends TableIf> table = 
databaseIf.get().getTable(analysisInfo.tblId);
+                    row.add(table.isPresent() ? table.get().getName() : "Table 
may get deleted");
+                } else {
+                    row.add("DB may get deleted");
+                }
+                row.add(analysisInfo.colName);
+                row.add(analysisInfo.jobType.toString());
+                row.add(analysisInfo.analysisType.toString());
+                row.add(analysisInfo.message);
+                row.add(TimeUtils.getDatetimeFormatWithTimeZone().format(
+                        
LocalDateTime.ofInstant(Instant.ofEpochMilli(analysisInfo.lastExecTimeInMs),
+                                ZoneId.systemDefault())));
+                row.add(analysisInfo.state.toString());
+                
row.add(Env.getCurrentEnv().getAnalysisManager().getJobProgress(analysisInfo.jobId));
+                row.add(analysisInfo.scheduleType.toString());
+                LocalDateTime startTime =
+                        
LocalDateTime.ofInstant(Instant.ofEpochMilli(analysisInfo.startTime),
+                                java.time.ZoneId.systemDefault());
+                LocalDateTime endTime =
+                        
LocalDateTime.ofInstant(Instant.ofEpochMilli(analysisInfo.endTime),
+                                java.time.ZoneId.systemDefault());
+                row.add(startTime.format(formatter));
+                row.add(endTime.format(formatter));
+                row.add(analysisInfo.priority.name());
+                row.add(String.valueOf(analysisInfo.enablePartition));
+                resultRows.add(row);
+            } catch (Exception e) {
+                LOG.warn("Failed to get analyze info for table {}.{}.{}, 
reason: {}",
+                        analysisInfo.catalogId, analysisInfo.dbId, 
analysisInfo.tblId, e.getMessage());
+            }
+        }
+        return new ShowResultSet(META_DATA, resultRows);
+    }
+
+    @Override
+    public RedirectStatus toRedirectStatus() {
+        return RedirectStatus.FORWARD_NO_SYNC;
+    }
+}
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/CommandVisitor.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/CommandVisitor.java
index bbbff372d58..7b5c762dded 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/CommandVisitor.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/CommandVisitor.java
@@ -91,6 +91,7 @@ import 
org.apache.doris.nereids.trees.plans.commands.SetDefaultStorageVaultComma
 import org.apache.doris.nereids.trees.plans.commands.SetOptionsCommand;
 import org.apache.doris.nereids.trees.plans.commands.SetTransactionCommand;
 import org.apache.doris.nereids.trees.plans.commands.SetUserPropertiesCommand;
+import org.apache.doris.nereids.trees.plans.commands.ShowAnalyzeCommand;
 import org.apache.doris.nereids.trees.plans.commands.ShowAuthorsCommand;
 import org.apache.doris.nereids.trees.plans.commands.ShowBackendsCommand;
 import org.apache.doris.nereids.trees.plans.commands.ShowBrokerCommand;
@@ -773,4 +774,8 @@ public interface CommandVisitor<R, C> {
                                           C context) {
         return visitCommand(alterRepositoryCommand, context);
     }
+
+    default R visitShowAnalyzeCommand(ShowAnalyzeCommand showAnalyzeCommand, C 
context) {
+        return visitCommand(showAnalyzeCommand, context);
+    }
 }
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/statistics/AnalysisManager.java 
b/fe/fe-core/src/main/java/org/apache/doris/statistics/AnalysisManager.java
index d92179cf2f0..8501bc0c121 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/statistics/AnalysisManager.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/statistics/AnalysisManager.java
@@ -578,22 +578,35 @@ public class AnalysisManager implements Writable {
     }
 
     public List<AnalysisInfo> findAnalysisJobs(ShowAnalyzeStmt stmt) {
-        String state = stmt.getStateValue();
-        TableName tblName = stmt.getDbTableName();
+        String ctl = null;
+        String db = null;
+        String table = null;
+        TableName dbTableName = stmt.getDbTableName();
+        if (dbTableName != null) {
+            ctl = dbTableName.getCtl();
+            db = dbTableName.getDb();
+            table = dbTableName.getTbl();
+        }
+        return findAnalysisJobs(stmt.getStateValue(), ctl, db, table, 
stmt.getJobId(), stmt.isAuto());
+    }
+
+    public List<AnalysisInfo> findAnalysisJobs(String state, String ctl, 
String db,
+            String table, long jobId, boolean isAuto) {
         TableIf tbl = null;
-        if (tblName != null) {
-            tbl = StatisticsUtil.findTable(tblName.getCtl(), tblName.getDb(), 
tblName.getTbl());
+        boolean tableSpecified = ctl != null && db != null && table != null;
+        if (tableSpecified) {
+            tbl = StatisticsUtil.findTable(ctl, db, table);
         }
         long tblId = tbl == null ? -1 : tbl.getId();
         synchronized (analysisJobInfoMap) {
             return analysisJobInfoMap.values().stream()
-                .filter(a -> stmt.getJobId() == 0 || a.jobId == 
stmt.getJobId())
-                .filter(a -> state == null || 
a.state.equals(AnalysisState.valueOf(state)))
-                .filter(a -> tblName == null || a.tblId == tblId)
-                .filter(a -> stmt.isAuto() && a.jobType.equals(JobType.SYSTEM)
-                    || !stmt.isAuto() && a.jobType.equals(JobType.MANUAL))
-                .sorted(Comparator.comparingLong(a -> a.jobId))
-                .collect(Collectors.toList());
+                    .filter(a -> jobId == 0 || a.jobId == jobId)
+                    .filter(a -> state == null || 
a.state.equals(AnalysisState.valueOf(state.toUpperCase())))
+                    .filter(a -> !tableSpecified || a.tblId == tblId)
+                    .filter(a -> isAuto && a.jobType.equals(JobType.SYSTEM)
+                            || !isAuto && a.jobType.equals(JobType.MANUAL))
+                    .sorted(Comparator.comparingLong(a -> a.jobId))
+                    .collect(Collectors.toList());
         }
     }
 
diff --git a/regression-test/suites/statistics/analyze_stats.groovy 
b/regression-test/suites/statistics/analyze_stats.groovy
index 1078cbf218a..e57bef056fc 100644
--- a/regression-test/suites/statistics/analyze_stats.groovy
+++ b/regression-test/suites/statistics/analyze_stats.groovy
@@ -2711,6 +2711,29 @@ PARTITION `p599` VALUES IN (599)
     assertEquals("\'name1\'", result[0][7])
     assertEquals("\'name3\'", result[0][8])
 
+    // Test show analyze
+    sql """drop stats region"""
+    sql """analyze table region"""
+    result = sql """show analyze region"""
+    assertEquals(1, result.size())
+    result = sql """show auto analyze region"""
+    assertEquals(0, result.size())
+    for (int i = 0; i < 100; i++) {
+        result = sql """show analyze region"""
+        if (!result[0][9].equals("FINISHED")) {
+            Thread.sleep(1000)
+            continue;
+        }
+        break;
+    }
+    assertEquals("FINISHED", result[0][9])
+    assertEquals(1, result.size())
+    result = sql """show analyze region where state="FINISHED";"""
+    assertEquals(1, result.size())
+    result = sql """show analyze region where state="pending";"""
+    assertEquals(0, result.size())
+
+
     // Test sample string type min max
     sql """
      CREATE TABLE `string_min_max` (
@@ -2957,4 +2980,3 @@ PARTITION `p599` VALUES IN (599)
 
     sql """drop database if exists test_version"""
 }
-


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

Reply via email to