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