risingwave_meta/controller/catalog/
mod.rs

1// Copyright 2025 RisingWave Labs
2//
3// Licensed under the Apache License, Version 2.0 (the "License");
4// you may not use this file except in compliance with the License.
5// You may obtain a copy of the License at
6//
7//     http://www.apache.org/licenses/LICENSE-2.0
8//
9// Unless required by applicable law or agreed to in writing, software
10// distributed under the License is distributed on an "AS IS" BASIS,
11// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
12// See the License for the specific language governing permissions and
13// limitations under the License.
14
15mod alter_op;
16mod create_op;
17mod drop_op;
18mod get_op;
19mod list_op;
20mod test;
21mod util;
22
23use std::collections::{BTreeSet, HashMap, HashSet};
24use std::iter;
25use std::mem::take;
26use std::sync::Arc;
27
28use anyhow::anyhow;
29use itertools::Itertools;
30use risingwave_common::catalog::{
31    DEFAULT_SCHEMA_NAME, FragmentTypeFlag, SYSTEM_SCHEMAS, TableOption,
32};
33use risingwave_common::current_cluster_version;
34use risingwave_common::id::JobId;
35use risingwave_common::secret::LocalSecretManager;
36use risingwave_common::util::stream_graph_visitor::visit_stream_node_cont_mut;
37use risingwave_connector::source::UPSTREAM_SOURCE_KEY;
38use risingwave_connector::source::cdc::build_cdc_table_id;
39use risingwave_meta_model::object::ObjectType;
40use risingwave_meta_model::prelude::*;
41use risingwave_meta_model::table::TableType;
42use risingwave_meta_model::{
43    ActorId, ColumnCatalogArray, ConnectionId, CreateType, DatabaseId, FragmentId, I32Array,
44    IndexId, JobStatus, ObjectId, Property, SchemaId, SecretId, SinkFormatDesc, SinkId, SourceId,
45    StreamNode, StreamSourceInfo, StreamingParallelism, SubscriptionId, TableId, UserId, ViewId,
46    connection, database, fragment, function, index, object, object_dependency, schema, secret,
47    sink, source, streaming_job, subscription, table, user_privilege, view,
48};
49use risingwave_pb::catalog::connection::Info as ConnectionInfo;
50use risingwave_pb::catalog::subscription::SubscriptionState;
51use risingwave_pb::catalog::table::PbTableType;
52use risingwave_pb::catalog::{
53    PbComment, PbConnection, PbDatabase, PbFunction, PbIndex, PbSchema, PbSecret, PbSink, PbSource,
54    PbStreamJobStatus, PbSubscription, PbTable, PbView,
55};
56use risingwave_pb::meta::cancel_creating_jobs_request::PbCreatingJobInfo;
57use risingwave_pb::meta::list_object_dependencies_response::PbObjectDependencies;
58use risingwave_pb::meta::object::PbObjectInfo;
59use risingwave_pb::meta::subscribe_response::{
60    Info as NotificationInfo, Info, Operation as NotificationOperation, Operation,
61};
62use risingwave_pb::meta::{PbObject, PbObjectGroup};
63use risingwave_pb::stream_plan::stream_node::NodeBody;
64use risingwave_pb::telemetry::PbTelemetryEventStage;
65use risingwave_pb::user::PbUserInfo;
66use sea_orm::ActiveValue::Set;
67use sea_orm::sea_query::{Expr, Query, SimpleExpr};
68use sea_orm::{
69    ActiveModelTrait, ColumnTrait, DatabaseConnection, DatabaseTransaction, EntityTrait,
70    IntoActiveModel, JoinType, PaginatorTrait, QueryFilter, QuerySelect, RelationTrait,
71    SelectColumns, TransactionTrait, Value,
72};
73use tokio::sync::oneshot::Sender;
74use tokio::sync::{RwLock, RwLockReadGuard, RwLockWriteGuard};
75use tracing::info;
76
77use super::utils::{
78    check_subscription_name_duplicate, get_internal_tables_by_id, rename_relation,
79    rename_relation_refer,
80};
81use crate::controller::ObjectModel;
82use crate::controller::catalog::util::update_internal_tables;
83use crate::controller::utils::*;
84use crate::manager::{
85    IGNORED_NOTIFICATION_VERSION, MetaSrvEnv, NotificationVersion,
86    get_referred_connection_ids_from_source, get_referred_secret_ids_from_source,
87};
88use crate::rpc::ddl_controller::DropMode;
89use crate::telemetry::{MetaTelemetryJobDesc, report_event};
90use crate::{MetaError, MetaResult};
91
92pub type Catalog = (
93    Vec<PbDatabase>,
94    Vec<PbSchema>,
95    Vec<PbTable>,
96    Vec<PbSource>,
97    Vec<PbSink>,
98    Vec<PbSubscription>,
99    Vec<PbIndex>,
100    Vec<PbView>,
101    Vec<PbFunction>,
102    Vec<PbConnection>,
103    Vec<PbSecret>,
104);
105
106pub type CatalogControllerRef = Arc<CatalogController>;
107
108/// `CatalogController` is the controller for catalog related operations, including database, schema, table, view, etc.
109pub struct CatalogController {
110    pub(crate) env: MetaSrvEnv,
111    pub(crate) inner: RwLock<CatalogControllerInner>,
112}
113
114#[derive(Clone, Default, Debug)]
115pub struct DropTableConnectorContext {
116    // we only apply one drop connector action for one table each time, so no need to vector here
117    pub(crate) to_change_streaming_job_id: JobId,
118    pub(crate) to_remove_state_table_id: TableId,
119    pub(crate) to_remove_source_id: SourceId,
120}
121
122#[derive(Clone, Default, Debug)]
123pub struct ReleaseContext {
124    pub(crate) database_id: DatabaseId,
125    pub(crate) removed_streaming_job_ids: Vec<JobId>,
126    /// Dropped state table list, need to unregister from hummock.
127    pub(crate) removed_state_table_ids: Vec<TableId>,
128
129    /// Dropped secrets, need to remove from secret manager.
130    pub(crate) removed_secret_ids: Vec<SecretId>,
131    /// Dropped sources (when `DROP SOURCE`), need to unregister from source manager.
132    pub(crate) removed_source_ids: Vec<SourceId>,
133    /// Dropped Source fragments (when `DROP MATERIALIZED VIEW` referencing sources),
134    /// need to unregister from source manager.
135    pub(crate) removed_source_fragments: HashMap<SourceId, BTreeSet<FragmentId>>,
136
137    pub(crate) removed_actors: HashSet<ActorId>,
138    pub(crate) removed_fragments: HashSet<FragmentId>,
139
140    /// Removed sink fragment by target fragment.
141    pub(crate) removed_sink_fragment_by_targets: HashMap<FragmentId, Vec<FragmentId>>,
142
143    /// Dropped iceberg table sinks
144    pub(crate) removed_iceberg_table_sinks: Vec<PbSink>,
145}
146
147impl CatalogController {
148    pub async fn new(env: MetaSrvEnv) -> MetaResult<Self> {
149        let meta_store = env.meta_store();
150        let catalog_controller = Self {
151            env,
152            inner: RwLock::new(CatalogControllerInner {
153                db: meta_store.conn,
154                creating_table_finish_notifier: HashMap::new(),
155                dropped_tables: HashMap::new(),
156            }),
157        };
158
159        catalog_controller.init().await?;
160        Ok(catalog_controller)
161    }
162
163    /// Used in `NotificationService::subscribe`.
164    /// Need to pay attention to the order of acquiring locks to prevent deadlock problems.
165    pub async fn get_inner_read_guard(&self) -> RwLockReadGuard<'_, CatalogControllerInner> {
166        self.inner.read().await
167    }
168
169    pub async fn get_inner_write_guard(&self) -> RwLockWriteGuard<'_, CatalogControllerInner> {
170        self.inner.write().await
171    }
172}
173
174pub struct CatalogControllerInner {
175    pub(crate) db: DatabaseConnection,
176    /// Registered finish notifiers for creating tables.
177    ///
178    /// `DdlController` will update this map, and pass the `tx` side to `CatalogController`.
179    /// On notifying, we can remove the entry from this map.
180    #[expect(clippy::type_complexity)]
181    pub creating_table_finish_notifier:
182        HashMap<DatabaseId, HashMap<JobId, Vec<Sender<Result<NotificationVersion, String>>>>>,
183    /// Tables have been dropped from the meta store, but the corresponding barrier remains unfinished.
184    pub dropped_tables: HashMap<TableId, PbTable>,
185}
186
187impl CatalogController {
188    pub(crate) async fn notify_frontend(
189        &self,
190        operation: NotificationOperation,
191        info: NotificationInfo,
192    ) -> NotificationVersion {
193        self.env
194            .notification_manager()
195            .notify_frontend(operation, info)
196            .await
197    }
198
199    pub(crate) async fn notify_frontend_relation_info(
200        &self,
201        operation: NotificationOperation,
202        relation_info: PbObjectInfo,
203    ) -> NotificationVersion {
204        self.env
205            .notification_manager()
206            .notify_frontend_object_info(operation, relation_info)
207            .await
208    }
209
210    pub(crate) async fn current_notification_version(&self) -> NotificationVersion {
211        self.env.notification_manager().current_version().await
212    }
213}
214
215impl CatalogController {
216    pub async fn finish_create_subscription_catalog(
217        &self,
218        subscription_id: SubscriptionId,
219    ) -> MetaResult<()> {
220        let inner = self.inner.write().await;
221        let txn = inner.db.begin().await?;
222
223        // update `created_at` as now() and `created_at_cluster_version` as current cluster version.
224        let res = Object::update_many()
225            .col_expr(object::Column::CreatedAt, Expr::current_timestamp().into())
226            .col_expr(
227                object::Column::CreatedAtClusterVersion,
228                current_cluster_version().into(),
229            )
230            .filter(object::Column::Oid.eq(subscription_id))
231            .exec(&txn)
232            .await?;
233        if res.rows_affected == 0 {
234            return Err(MetaError::catalog_id_not_found(
235                "subscription",
236                subscription_id,
237            ));
238        }
239
240        // mark the target subscription as `Create`.
241        let job = subscription::ActiveModel {
242            subscription_id: Set(subscription_id),
243            subscription_state: Set(SubscriptionState::Created.into()),
244            ..Default::default()
245        };
246        job.update(&txn).await?;
247
248        let _ = grant_default_privileges_automatically(&txn, subscription_id).await?;
249
250        txn.commit().await?;
251
252        Ok(())
253    }
254
255    pub async fn notify_create_subscription(
256        &self,
257        subscription_id: SubscriptionId,
258    ) -> MetaResult<NotificationVersion> {
259        let inner = self.inner.read().await;
260        let (subscription, obj) = Subscription::find_by_id(subscription_id)
261            .find_also_related(Object)
262            .filter(subscription::Column::SubscriptionState.eq(SubscriptionState::Created as i32))
263            .one(&inner.db)
264            .await?
265            .ok_or_else(|| MetaError::catalog_id_not_found("subscription", subscription_id))?;
266
267        let mut version = self
268            .notify_frontend(
269                NotificationOperation::Add,
270                NotificationInfo::ObjectGroup(PbObjectGroup {
271                    objects: vec![PbObject {
272                        object_info: PbObjectInfo::Subscription(
273                            ObjectModel(subscription, obj.unwrap()).into(),
274                        )
275                        .into(),
276                    }],
277                }),
278            )
279            .await;
280
281        // notify default privileges about the new subscription
282        let updated_user_ids: Vec<UserId> = UserPrivilege::find()
283            .select_only()
284            .distinct()
285            .column(user_privilege::Column::UserId)
286            .filter(user_privilege::Column::Oid.eq(subscription_id.as_object_id()))
287            .into_tuple()
288            .all(&inner.db)
289            .await?;
290
291        if !updated_user_ids.is_empty() {
292            let updated_user_infos = list_user_info_by_ids(updated_user_ids, &inner.db).await?;
293            version = self.notify_users_update(updated_user_infos).await;
294        }
295
296        Ok(version)
297    }
298
299    // for telemetry
300    pub async fn get_connector_usage(&self) -> MetaResult<jsonbb::Value> {
301        // get connector usage by source/sink
302        // the expect format is like:
303        // {
304        //     "source": [{
305        //         "$source_id": {
306        //             "connector": "kafka",
307        //             "format": "plain",
308        //             "encode": "json"
309        //         },
310        //     }],
311        //     "sink": [{
312        //         "$sink_id": {
313        //             "connector": "pulsar",
314        //             "format": "upsert",
315        //             "encode": "avro"
316        //         },
317        //     }],
318        // }
319
320        let inner = self.inner.read().await;
321        let source_props_and_info: Vec<(i32, Property, Option<StreamSourceInfo>)> = Source::find()
322            .select_only()
323            .column(source::Column::SourceId)
324            .column(source::Column::WithProperties)
325            .column(source::Column::SourceInfo)
326            .into_tuple()
327            .all(&inner.db)
328            .await?;
329        let sink_props_and_info: Vec<(i32, Property, Option<SinkFormatDesc>)> = Sink::find()
330            .select_only()
331            .column(sink::Column::SinkId)
332            .column(sink::Column::Properties)
333            .column(sink::Column::SinkFormatDesc)
334            .into_tuple()
335            .all(&inner.db)
336            .await?;
337        drop(inner);
338
339        let get_connector_from_property = |property: &Property| -> String {
340            property
341                .0
342                .get(UPSTREAM_SOURCE_KEY)
343                .cloned()
344                .unwrap_or_default()
345        };
346
347        let source_report: Vec<jsonbb::Value> = source_props_and_info
348            .iter()
349            .map(|(oid, property, info)| {
350                let connector_name = get_connector_from_property(property);
351                let mut format = None;
352                let mut encode = None;
353                if let Some(info) = info {
354                    let pb_info = info.to_protobuf();
355                    format = Some(pb_info.format().as_str_name());
356                    encode = Some(pb_info.row_encode().as_str_name());
357                }
358                jsonbb::json!({
359                    oid.to_string(): {
360                        "connector": connector_name,
361                        "format": format,
362                        "encode": encode,
363                    },
364                })
365            })
366            .collect_vec();
367
368        let sink_report: Vec<jsonbb::Value> = sink_props_and_info
369            .iter()
370            .map(|(oid, property, info)| {
371                let connector_name = get_connector_from_property(property);
372                let mut format = None;
373                let mut encode = None;
374                if let Some(info) = info {
375                    let pb_info = info.to_protobuf();
376                    format = Some(pb_info.format().as_str_name());
377                    encode = Some(pb_info.encode().as_str_name());
378                }
379                jsonbb::json!({
380                    oid.to_string(): {
381                        "connector": connector_name,
382                        "format": format,
383                        "encode": encode,
384                    },
385                })
386            })
387            .collect_vec();
388
389        Ok(jsonbb::json!({
390                "source": source_report,
391                "sink": sink_report,
392        }))
393    }
394
395    pub async fn clean_dirty_subscription(
396        &self,
397        database_id: Option<DatabaseId>,
398    ) -> MetaResult<()> {
399        let inner = self.inner.write().await;
400        let txn = inner.db.begin().await?;
401        let filter_condition = object::Column::ObjType.eq(ObjectType::Subscription).and(
402            object::Column::Oid.not_in_subquery(
403                Query::select()
404                    .column(subscription::Column::SubscriptionId)
405                    .from(Subscription)
406                    .and_where(
407                        subscription::Column::SubscriptionState
408                            .eq(SubscriptionState::Created as i32),
409                    )
410                    .take(),
411            ),
412        );
413
414        let filter_condition = if let Some(database_id) = database_id {
415            filter_condition.and(object::Column::DatabaseId.eq(database_id))
416        } else {
417            filter_condition
418        };
419        Object::delete_many()
420            .filter(filter_condition)
421            .exec(&txn)
422            .await?;
423        txn.commit().await?;
424        // We don't need to notify the frontend, because the Init subscription is not send to frontend.
425        Ok(())
426    }
427
428    /// `clean_dirty_creating_jobs` cleans up creating jobs that are creating in Foreground mode or in Initial status.
429    pub async fn clean_dirty_creating_jobs(
430        &self,
431        database_id: Option<DatabaseId>,
432    ) -> MetaResult<Vec<SourceId>> {
433        let inner = self.inner.write().await;
434        let txn = inner.db.begin().await?;
435
436        let filter_condition = streaming_job::Column::JobStatus.eq(JobStatus::Initial).or(
437            streaming_job::Column::JobStatus
438                .eq(JobStatus::Creating)
439                .and(streaming_job::Column::CreateType.eq(CreateType::Foreground)),
440        );
441
442        let filter_condition = if let Some(database_id) = database_id {
443            filter_condition.and(object::Column::DatabaseId.eq(database_id))
444        } else {
445            filter_condition
446        };
447
448        let mut dirty_job_objs: Vec<PartialObject> = streaming_job::Entity::find()
449            .select_only()
450            .column(streaming_job::Column::JobId)
451            .columns([
452                object::Column::Oid,
453                object::Column::ObjType,
454                object::Column::SchemaId,
455                object::Column::DatabaseId,
456            ])
457            .join(JoinType::InnerJoin, streaming_job::Relation::Object.def())
458            .filter(filter_condition)
459            .into_partial_model()
460            .all(&txn)
461            .await?;
462
463        // Check if there are any pending iceberg table jobs.
464        let dirty_iceberg_jobs = find_dirty_iceberg_table_jobs(&txn, database_id).await?;
465        if !dirty_iceberg_jobs.is_empty() {
466            dirty_job_objs.extend(dirty_iceberg_jobs);
467        }
468
469        Self::clean_dirty_sink_downstreams(&txn).await?;
470
471        if dirty_job_objs.is_empty() {
472            return Ok(vec![]);
473        }
474
475        self.log_cleaned_dirty_jobs(&dirty_job_objs, &txn).await?;
476
477        let dirty_job_ids = dirty_job_objs.iter().map(|obj| obj.oid).collect::<Vec<_>>();
478
479        // Filter out dummy objs for replacement.
480        // todo: we'd better introduce a new dummy object type for replacement.
481        let all_dirty_table_ids = dirty_job_objs
482            .iter()
483            .filter(|obj| obj.obj_type == ObjectType::Table)
484            .map(|obj| obj.oid)
485            .collect_vec();
486        let dirty_table_type_map: HashMap<ObjectId, TableType> = Table::find()
487            .select_only()
488            .column(table::Column::TableId)
489            .column(table::Column::TableType)
490            .filter(table::Column::TableId.is_in(all_dirty_table_ids))
491            .into_tuple::<(ObjectId, TableType)>()
492            .all(&txn)
493            .await?
494            .into_iter()
495            .collect();
496
497        let dirty_background_jobs: HashSet<ObjectId> = streaming_job::Entity::find()
498            .select_only()
499            .column(streaming_job::Column::JobId)
500            .filter(
501                streaming_job::Column::JobId
502                    .is_in(dirty_job_ids.clone())
503                    .and(streaming_job::Column::CreateType.eq(CreateType::Background)),
504            )
505            .into_tuple()
506            .all(&txn)
507            .await?
508            .into_iter()
509            .collect();
510
511        // notify delete for failed materialized views and background jobs.
512        let to_notify_objs = dirty_job_objs
513            .into_iter()
514            .filter(|obj| {
515                matches!(
516                    dirty_table_type_map.get(&obj.oid),
517                    Some(TableType::MaterializedView)
518                ) || dirty_background_jobs.contains(&obj.oid)
519            })
520            .collect_vec();
521
522        // The source ids for dirty tables with connector.
523        // FIXME: we should also clean dirty sources.
524        let dirty_associated_source_ids: Vec<SourceId> = Table::find()
525            .select_only()
526            .column(table::Column::OptionalAssociatedSourceId)
527            .filter(
528                table::Column::TableId
529                    .is_in(dirty_job_ids.clone())
530                    .and(table::Column::OptionalAssociatedSourceId.is_not_null()),
531            )
532            .into_tuple()
533            .all(&txn)
534            .await?;
535
536        let dirty_state_table_ids: Vec<TableId> = Table::find()
537            .select_only()
538            .column(table::Column::TableId)
539            .filter(table::Column::BelongsToJobId.is_in(dirty_job_ids.clone()))
540            .into_tuple()
541            .all(&txn)
542            .await?;
543
544        let dirty_internal_table_objs = Object::find()
545            .select_only()
546            .columns([
547                object::Column::Oid,
548                object::Column::ObjType,
549                object::Column::SchemaId,
550                object::Column::DatabaseId,
551            ])
552            .join(JoinType::InnerJoin, object::Relation::Table.def())
553            .filter(table::Column::BelongsToJobId.is_in(to_notify_objs.iter().map(|obj| obj.oid)))
554            .into_partial_model()
555            .all(&txn)
556            .await?;
557
558        let to_delete_objs: HashSet<ObjectId> = dirty_job_ids
559            .clone()
560            .into_iter()
561            .chain(
562                dirty_state_table_ids
563                    .into_iter()
564                    .map(|table_id| table_id.as_object_id()),
565            )
566            .chain(
567                dirty_associated_source_ids
568                    .iter()
569                    .map(|source_id| source_id.as_object_id()),
570            )
571            .collect();
572
573        let res = Object::delete_many()
574            .filter(object::Column::Oid.is_in(to_delete_objs))
575            .exec(&txn)
576            .await?;
577        assert!(res.rows_affected > 0);
578
579        txn.commit().await?;
580
581        let object_group = build_object_group_for_delete(
582            to_notify_objs
583                .into_iter()
584                .chain(dirty_internal_table_objs.into_iter())
585                .collect_vec(),
586        );
587
588        let _version = self
589            .notify_frontend(NotificationOperation::Delete, object_group)
590            .await;
591
592        Ok(dirty_associated_source_ids)
593    }
594
595    pub async fn comment_on(&self, comment: PbComment) -> MetaResult<NotificationVersion> {
596        let inner = self.inner.write().await;
597        let txn = inner.db.begin().await?;
598        ensure_object_id(ObjectType::Database, comment.database_id, &txn).await?;
599        ensure_object_id(ObjectType::Schema, comment.schema_id, &txn).await?;
600        let table_obj = Object::find_by_id(comment.table_id)
601            .one(&txn)
602            .await?
603            .ok_or_else(|| MetaError::catalog_id_not_found("table", comment.table_id))?;
604
605        let table = if let Some(col_idx) = comment.column_index {
606            let columns: ColumnCatalogArray = Table::find_by_id(comment.table_id)
607                .select_only()
608                .column(table::Column::Columns)
609                .into_tuple()
610                .one(&txn)
611                .await?
612                .ok_or_else(|| MetaError::catalog_id_not_found("table", comment.table_id))?;
613            let mut pb_columns = columns.to_protobuf();
614
615            let column = pb_columns
616                .get_mut(col_idx as usize)
617                .ok_or_else(|| MetaError::catalog_id_not_found("column", col_idx))?;
618            let column_desc = column.column_desc.as_mut().ok_or_else(|| {
619                anyhow!(
620                    "column desc at index {} for table id {} not found",
621                    col_idx,
622                    comment.table_id
623                )
624            })?;
625            column_desc.description = comment.description;
626            table::ActiveModel {
627                table_id: Set(comment.table_id),
628                columns: Set(pb_columns.into()),
629                ..Default::default()
630            }
631            .update(&txn)
632            .await?
633        } else {
634            table::ActiveModel {
635                table_id: Set(comment.table_id),
636                description: Set(comment.description),
637                ..Default::default()
638            }
639            .update(&txn)
640            .await?
641        };
642        txn.commit().await?;
643
644        let version = self
645            .notify_frontend_relation_info(
646                NotificationOperation::Update,
647                PbObjectInfo::Table(ObjectModel(table, table_obj).into()),
648            )
649            .await;
650
651        Ok(version)
652    }
653
654    async fn notify_hummock_dropped_tables(&self, tables: Vec<PbTable>) {
655        if tables.is_empty() {
656            return;
657        }
658        let objects = tables
659            .into_iter()
660            .map(|t| PbObject {
661                object_info: Some(PbObjectInfo::Table(t)),
662            })
663            .collect();
664        let group = NotificationInfo::ObjectGroup(PbObjectGroup { objects });
665        self.env
666            .notification_manager()
667            .notify_hummock(NotificationOperation::Delete, group.clone())
668            .await;
669        self.env
670            .notification_manager()
671            .notify_compactor(NotificationOperation::Delete, group)
672            .await;
673    }
674
675    pub async fn complete_dropped_tables(&self, table_ids: impl IntoIterator<Item = TableId>) {
676        let mut inner = self.inner.write().await;
677        let tables = inner.complete_dropped_tables(table_ids);
678        self.notify_hummock_dropped_tables(tables).await;
679    }
680
681    pub async fn cleanup_dropped_tables(&self) {
682        let mut inner = self.inner.write().await;
683        let tables = inner.dropped_tables.drain().map(|(_, t)| t).collect();
684        self.notify_hummock_dropped_tables(tables).await;
685    }
686
687    pub async fn stats(&self) -> MetaResult<CatalogStats> {
688        let inner = self.inner.read().await;
689
690        let mut table_num_map: HashMap<_, _> = Table::find()
691            .select_only()
692            .column(table::Column::TableType)
693            .column_as(table::Column::TableId.count(), "num")
694            .group_by(table::Column::TableType)
695            .having(table::Column::TableType.ne(TableType::Internal))
696            .into_tuple::<(TableType, i64)>()
697            .all(&inner.db)
698            .await?
699            .into_iter()
700            .map(|(table_type, num)| (table_type, num as u64))
701            .collect();
702
703        let source_num = Source::find().count(&inner.db).await?;
704        let sink_num = Sink::find().count(&inner.db).await?;
705        let function_num = Function::find().count(&inner.db).await?;
706        let streaming_job_num = StreamingJob::find().count(&inner.db).await?;
707
708        let actor_num = {
709            let guard = self.env.shared_actor_info.read_guard();
710            guard
711                .iter_over_fragments()
712                .map(|(_, fragment)| fragment.actors.len() as u64)
713                .sum::<u64>()
714        };
715
716        Ok(CatalogStats {
717            table_num: table_num_map.remove(&TableType::Table).unwrap_or(0),
718            mview_num: table_num_map
719                .remove(&TableType::MaterializedView)
720                .unwrap_or(0),
721            index_num: table_num_map.remove(&TableType::Index).unwrap_or(0),
722            source_num,
723            sink_num,
724            function_num,
725            streaming_job_num,
726            actor_num,
727        })
728    }
729}
730
731/// `CatalogStats` is a struct to store the statistics of all catalogs.
732pub struct CatalogStats {
733    pub table_num: u64,
734    pub mview_num: u64,
735    pub index_num: u64,
736    pub source_num: u64,
737    pub sink_num: u64,
738    pub function_num: u64,
739    pub streaming_job_num: u64,
740    pub actor_num: u64,
741}
742
743impl CatalogControllerInner {
744    pub async fn snapshot(&self) -> MetaResult<(Catalog, Vec<PbUserInfo>)> {
745        let databases = self.list_databases().await?;
746        let schemas = self.list_schemas().await?;
747        let tables = self.list_tables().await?;
748        let sources = self.list_sources().await?;
749        let sinks = self.list_sinks().await?;
750        let subscriptions = self.list_subscriptions().await?;
751        let indexes = self.list_indexes().await?;
752        let views = self.list_views().await?;
753        let functions = self.list_functions().await?;
754        let connections = self.list_connections().await?;
755        let secrets = self.list_secrets().await?;
756
757        let users = self.list_users().await?;
758
759        Ok((
760            (
761                databases,
762                schemas,
763                tables,
764                sources,
765                sinks,
766                subscriptions,
767                indexes,
768                views,
769                functions,
770                connections,
771                secrets,
772            ),
773            users,
774        ))
775    }
776
777    async fn list_databases(&self) -> MetaResult<Vec<PbDatabase>> {
778        let db_objs = Database::find()
779            .find_also_related(Object)
780            .all(&self.db)
781            .await?;
782        Ok(db_objs
783            .into_iter()
784            .map(|(db, obj)| ObjectModel(db, obj.unwrap()).into())
785            .collect())
786    }
787
788    async fn list_schemas(&self) -> MetaResult<Vec<PbSchema>> {
789        let schema_objs = Schema::find()
790            .find_also_related(Object)
791            .all(&self.db)
792            .await?;
793
794        Ok(schema_objs
795            .into_iter()
796            .map(|(schema, obj)| ObjectModel(schema, obj.unwrap()).into())
797            .collect())
798    }
799
800    async fn list_users(&self) -> MetaResult<Vec<PbUserInfo>> {
801        let mut user_infos: Vec<PbUserInfo> = User::find()
802            .all(&self.db)
803            .await?
804            .into_iter()
805            .map(Into::into)
806            .collect();
807
808        for user_info in &mut user_infos {
809            user_info.grant_privileges = get_user_privilege(user_info.id as _, &self.db).await?;
810        }
811        Ok(user_infos)
812    }
813
814    /// `list_all_tables` return all tables and internal tables.
815    pub async fn list_all_state_tables(&self) -> MetaResult<Vec<PbTable>> {
816        let table_objs = Table::find()
817            .find_also_related(Object)
818            .all(&self.db)
819            .await?;
820
821        Ok(table_objs
822            .into_iter()
823            .map(|(table, obj)| ObjectModel(table, obj.unwrap()).into())
824            .collect())
825    }
826
827    /// `list_tables` return all `CREATED` tables, `CREATING` materialized views/ `BACKGROUND` jobs and internal tables that belong to them.
828    async fn list_tables(&self) -> MetaResult<Vec<PbTable>> {
829        let table_objs = Table::find()
830            .find_also_related(Object)
831            .join(JoinType::LeftJoin, object::Relation::StreamingJob.def())
832            .filter(
833                streaming_job::Column::JobStatus.eq(JobStatus::Created).or(
834                    table::Column::TableType
835                        .eq(TableType::MaterializedView)
836                        .or(streaming_job::Column::CreateType.eq(CreateType::Background)),
837                ),
838            )
839            .all(&self.db)
840            .await?;
841
842        let job_statuses: HashMap<JobId, JobStatus> = StreamingJob::find()
843            .select_only()
844            .column(streaming_job::Column::JobId)
845            .column(streaming_job::Column::JobStatus)
846            .filter(
847                streaming_job::Column::JobStatus
848                    .eq(JobStatus::Created)
849                    .or(streaming_job::Column::CreateType.eq(CreateType::Background)),
850            )
851            .into_tuple::<(JobId, JobStatus)>()
852            .all(&self.db)
853            .await?
854            .into_iter()
855            .collect();
856
857        let job_ids: HashSet<JobId> = table_objs
858            .iter()
859            .map(|(t, _)| t.table_id.as_job_id())
860            .chain(job_statuses.keys().cloned())
861            .collect();
862
863        let internal_table_objs = Table::find()
864            .find_also_related(Object)
865            .filter(
866                table::Column::TableType
867                    .eq(TableType::Internal)
868                    .and(table::Column::BelongsToJobId.is_in(job_ids)),
869            )
870            .all(&self.db)
871            .await?;
872
873        Ok(table_objs
874            .into_iter()
875            .chain(internal_table_objs.into_iter())
876            .map(|(table, obj)| {
877                // Correctly set the stream job status for creating materialized views and internal tables.
878                // If the table is not contained in `job_statuses`, it means the job is a creating mv.
879                let status: PbStreamJobStatus = if table.table_type == TableType::Internal {
880                    (*job_statuses
881                        .get(&table.belongs_to_job_id.unwrap())
882                        .unwrap_or(&JobStatus::Creating))
883                    .into()
884                } else {
885                    (*job_statuses
886                        .get(&table.table_id.as_job_id())
887                        .unwrap_or(&JobStatus::Creating))
888                    .into()
889                };
890                let mut pb_table: PbTable = ObjectModel(table, obj.unwrap()).into();
891                pb_table.stream_job_status = status.into();
892                pb_table
893            })
894            .collect())
895    }
896
897    /// `list_sources` return all sources and `CREATED` ones if contains any streaming jobs.
898    async fn list_sources(&self) -> MetaResult<Vec<PbSource>> {
899        let mut source_objs = Source::find()
900            .find_also_related(Object)
901            .join(JoinType::LeftJoin, object::Relation::StreamingJob.def())
902            .filter(
903                streaming_job::Column::JobStatus
904                    .is_null()
905                    .or(streaming_job::Column::JobStatus.eq(JobStatus::Created)),
906            )
907            .all(&self.db)
908            .await?;
909
910        // filter out inner connector sources that are still under creating.
911        let created_table_ids: HashSet<TableId> = Table::find()
912            .select_only()
913            .column(table::Column::TableId)
914            .join(JoinType::InnerJoin, table::Relation::Object1.def())
915            .join(JoinType::LeftJoin, object::Relation::StreamingJob.def())
916            .filter(
917                table::Column::OptionalAssociatedSourceId
918                    .is_not_null()
919                    .and(streaming_job::Column::JobStatus.eq(JobStatus::Created)),
920            )
921            .into_tuple()
922            .all(&self.db)
923            .await?
924            .into_iter()
925            .collect();
926        source_objs.retain_mut(|(source, _)| {
927            source.optional_associated_table_id.is_none()
928                || created_table_ids.contains(&source.optional_associated_table_id.unwrap())
929        });
930
931        Ok(source_objs
932            .into_iter()
933            .map(|(source, obj)| ObjectModel(source, obj.unwrap()).into())
934            .collect())
935    }
936
937    /// `list_sinks` return all `CREATED` and `BACKGROUND` sinks.
938    async fn list_sinks(&self) -> MetaResult<Vec<PbSink>> {
939        let sink_objs = Sink::find()
940            .find_also_related(Object)
941            .join(JoinType::LeftJoin, object::Relation::StreamingJob.def())
942            .filter(
943                streaming_job::Column::JobStatus
944                    .eq(JobStatus::Created)
945                    .or(streaming_job::Column::CreateType.eq(CreateType::Background)),
946            )
947            .all(&self.db)
948            .await?;
949
950        let creating_sinks: HashSet<_> = StreamingJob::find()
951            .select_only()
952            .column(streaming_job::Column::JobId)
953            .filter(
954                streaming_job::Column::JobStatus
955                    .eq(JobStatus::Creating)
956                    .and(
957                        streaming_job::Column::JobId
958                            .is_in(sink_objs.iter().map(|(sink, _)| sink.sink_id)),
959                    ),
960            )
961            .into_tuple::<SinkId>()
962            .all(&self.db)
963            .await?
964            .into_iter()
965            .collect();
966
967        Ok(sink_objs
968            .into_iter()
969            .map(|(sink, obj)| {
970                let is_creating = creating_sinks.contains(&sink.sink_id);
971                let mut pb_sink: PbSink = ObjectModel(sink, obj.unwrap()).into();
972                pb_sink.stream_job_status = if is_creating {
973                    PbStreamJobStatus::Creating.into()
974                } else {
975                    PbStreamJobStatus::Created.into()
976                };
977                pb_sink
978            })
979            .collect())
980    }
981
982    /// `list_subscriptions` return all `CREATED` subscriptions.
983    async fn list_subscriptions(&self) -> MetaResult<Vec<PbSubscription>> {
984        let subscription_objs = Subscription::find()
985            .find_also_related(Object)
986            .filter(subscription::Column::SubscriptionState.eq(SubscriptionState::Created as i32))
987            .all(&self.db)
988            .await?;
989
990        Ok(subscription_objs
991            .into_iter()
992            .map(|(subscription, obj)| ObjectModel(subscription, obj.unwrap()).into())
993            .collect())
994    }
995
996    async fn list_views(&self) -> MetaResult<Vec<PbView>> {
997        let view_objs = View::find().find_also_related(Object).all(&self.db).await?;
998
999        Ok(view_objs
1000            .into_iter()
1001            .map(|(view, obj)| ObjectModel(view, obj.unwrap()).into())
1002            .collect())
1003    }
1004
1005    /// `list_indexes` return all `CREATED` and `BACKGROUND` indexes.
1006    async fn list_indexes(&self) -> MetaResult<Vec<PbIndex>> {
1007        let index_objs = Index::find()
1008            .find_also_related(Object)
1009            .join(JoinType::LeftJoin, object::Relation::StreamingJob.def())
1010            .filter(
1011                streaming_job::Column::JobStatus
1012                    .eq(JobStatus::Created)
1013                    .or(streaming_job::Column::CreateType.eq(CreateType::Background)),
1014            )
1015            .all(&self.db)
1016            .await?;
1017
1018        let creating_indexes: HashSet<_> = StreamingJob::find()
1019            .select_only()
1020            .column(streaming_job::Column::JobId)
1021            .filter(
1022                streaming_job::Column::JobStatus
1023                    .eq(JobStatus::Creating)
1024                    .and(
1025                        streaming_job::Column::JobId
1026                            .is_in(index_objs.iter().map(|(index, _)| index.index_id)),
1027                    ),
1028            )
1029            .into_tuple::<IndexId>()
1030            .all(&self.db)
1031            .await?
1032            .into_iter()
1033            .collect();
1034
1035        Ok(index_objs
1036            .into_iter()
1037            .map(|(index, obj)| {
1038                let is_creating = creating_indexes.contains(&index.index_id);
1039                let mut pb_index: PbIndex = ObjectModel(index, obj.unwrap()).into();
1040                pb_index.stream_job_status = if is_creating {
1041                    PbStreamJobStatus::Creating.into()
1042                } else {
1043                    PbStreamJobStatus::Created.into()
1044                };
1045                pb_index
1046            })
1047            .collect())
1048    }
1049
1050    async fn list_connections(&self) -> MetaResult<Vec<PbConnection>> {
1051        let conn_objs = Connection::find()
1052            .find_also_related(Object)
1053            .all(&self.db)
1054            .await?;
1055
1056        Ok(conn_objs
1057            .into_iter()
1058            .map(|(conn, obj)| ObjectModel(conn, obj.unwrap()).into())
1059            .collect())
1060    }
1061
1062    pub async fn list_secrets(&self) -> MetaResult<Vec<PbSecret>> {
1063        let secret_objs = Secret::find()
1064            .find_also_related(Object)
1065            .all(&self.db)
1066            .await?;
1067        Ok(secret_objs
1068            .into_iter()
1069            .map(|(secret, obj)| ObjectModel(secret, obj.unwrap()).into())
1070            .collect())
1071    }
1072
1073    async fn list_functions(&self) -> MetaResult<Vec<PbFunction>> {
1074        let func_objs = Function::find()
1075            .find_also_related(Object)
1076            .all(&self.db)
1077            .await?;
1078
1079        Ok(func_objs
1080            .into_iter()
1081            .map(|(func, obj)| ObjectModel(func, obj.unwrap()).into())
1082            .collect())
1083    }
1084
1085    pub(crate) fn register_finish_notifier(
1086        &mut self,
1087        database_id: DatabaseId,
1088        id: JobId,
1089        sender: Sender<Result<NotificationVersion, String>>,
1090    ) {
1091        self.creating_table_finish_notifier
1092            .entry(database_id)
1093            .or_default()
1094            .entry(id)
1095            .or_default()
1096            .push(sender);
1097    }
1098
1099    pub(crate) async fn streaming_job_is_finished(&mut self, id: JobId) -> MetaResult<bool> {
1100        let status = StreamingJob::find()
1101            .select_only()
1102            .column(streaming_job::Column::JobStatus)
1103            .filter(streaming_job::Column::JobId.eq(id))
1104            .into_tuple::<JobStatus>()
1105            .one(&self.db)
1106            .await?;
1107
1108        status
1109            .map(|status| status == JobStatus::Created)
1110            .ok_or_else(|| {
1111                MetaError::catalog_id_not_found("streaming job", "may have been cancelled/dropped")
1112            })
1113    }
1114
1115    pub(crate) fn notify_finish_failed(&mut self, database_id: Option<DatabaseId>, err: String) {
1116        if let Some(database_id) = database_id {
1117            if let Some(creating_tables) = self.creating_table_finish_notifier.remove(&database_id)
1118            {
1119                for tx in creating_tables.into_values().flatten() {
1120                    let _ = tx.send(Err(err.clone()));
1121                }
1122            }
1123        } else {
1124            for tx in take(&mut self.creating_table_finish_notifier)
1125                .into_values()
1126                .flatten()
1127                .flat_map(|(_, txs)| txs.into_iter())
1128            {
1129                let _ = tx.send(Err(err.clone()));
1130            }
1131        }
1132    }
1133
1134    pub async fn list_time_travel_table_ids(&self) -> MetaResult<Vec<TableId>> {
1135        let table_ids: Vec<TableId> = Table::find()
1136            .select_only()
1137            .filter(table::Column::TableType.is_in(vec![
1138                TableType::Table,
1139                TableType::MaterializedView,
1140                TableType::Index,
1141            ]))
1142            .column(table::Column::TableId)
1143            .into_tuple()
1144            .all(&self.db)
1145            .await?;
1146        Ok(table_ids)
1147    }
1148
1149    /// Since the tables have been dropped from both meta store and streaming jobs, this method removes those table copies.
1150    /// Returns the removed table copies.
1151    pub(crate) fn complete_dropped_tables(
1152        &mut self,
1153        table_ids: impl IntoIterator<Item = TableId>,
1154    ) -> Vec<PbTable> {
1155        table_ids
1156            .into_iter()
1157            .filter_map(|table_id| {
1158                self.dropped_tables.remove(&table_id).map_or_else(
1159                    || {
1160                        tracing::warn!(%table_id, "table not found");
1161                        None
1162                    },
1163                    Some,
1164                )
1165            })
1166            .collect()
1167    }
1168}