Skip to content

Commit 845ed68

Browse files
authored
feat: add rw_internal_table_info to identity which streaming job the internal table belongs (#19642)
1 parent 3960994 commit 845ed68

File tree

14 files changed

+77
-2
lines changed

14 files changed

+77
-2
lines changed

proto/catalog.proto

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -470,6 +470,9 @@ message Table {
470470
// The information used by webhook source to validate the incoming data.
471471
optional WebhookSourceInfo webhook_info = 41;
472472

473+
// This field stores the job ID for internal tables.
474+
optional uint32 job_id = 42;
475+
473476
// Per-table catalog version, used by schema change. `None` for internal
474477
// tables and tests. Not to be confused with the global catalog version for
475478
// notification service.

src/frontend/src/catalog/system_catalog/rw_catalog/mod.rs

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -61,4 +61,5 @@ mod rw_worker_nodes;
6161
mod rw_actor_id_to_ddl;
6262
mod rw_actor_splits;
6363
mod rw_fragment_id_to_ddl;
64+
mod rw_internal_table_info;
6465
mod rw_worker_actor_count;

src/frontend/src/catalog/system_catalog/rw_catalog/rw_fragment_id_to_ddl.rs

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -15,7 +15,7 @@
1515
use risingwave_common::types::Fields;
1616
use risingwave_frontend_macro::system_catalog;
1717

18-
/// Provides a mapping from `actor_id` to its ddl info.
18+
/// Provides a mapping from `fragment_id` to its ddl info.
1919
#[system_catalog(
2020
view,
2121
"rw_catalog.rw_fragment_id_to_ddl",
Lines changed: 55 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -0,0 +1,55 @@
1+
// Copyright 2024 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+
15+
use risingwave_common::types::Fields;
16+
use risingwave_frontend_macro::system_catalog;
17+
18+
#[system_catalog(
19+
view,
20+
"rw_catalog.rw_internal_table_info",
21+
"WITH all_streaming_jobs AS (
22+
SELECT id, name, 'table' as job_type, schema_id, owner FROM rw_tables
23+
UNION ALL
24+
SELECT id, name, 'materialized view' as job_type, schema_id, owner FROM rw_materialized_views
25+
UNION ALL
26+
SELECT id, name, 'sink' as job_type, schema_id, owner FROM rw_sinks
27+
UNION ALL
28+
SELECT id, name, 'index' as job_type, schema_id, owner FROM rw_indexes
29+
UNION ALL
30+
SELECT id, name, 'source' as job_type, schema_id, owner FROM rw_sources WHERE is_shared = true
31+
)
32+
33+
SELECT i.id,
34+
i.name,
35+
j.id as job_id,
36+
j.name as job_name,
37+
j.job_type,
38+
s.name as schema_name,
39+
u.name as owner
40+
FROM rw_catalog.rw_internal_tables i
41+
JOIN all_streaming_jobs j ON i.job_id = j.id
42+
JOIN rw_catalog.rw_schemas s ON j.schema_id = s.id
43+
JOIN rw_catalog.rw_users u ON j.owner = u.id"
44+
)]
45+
#[derive(Fields)]
46+
struct RwInternalTableInfo {
47+
#[primary_key]
48+
id: i32,
49+
name: String,
50+
job_id: i32,
51+
job_name: String,
52+
job_type: String,
53+
schema_name: String,
54+
owner: String,
55+
}

src/frontend/src/catalog/system_catalog/rw_catalog/rw_internal_tables.rs

Lines changed: 2 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -25,6 +25,7 @@ struct RwInternalTable {
2525
id: i32,
2626
name: String,
2727
schema_id: i32,
28+
job_id: i32,
2829
owner: i32,
2930
definition: String,
3031
acl: Vec<String>,
@@ -48,6 +49,7 @@ fn read_rw_internal_tables(reader: &SysCatalogReaderImpl) -> Result<Vec<RwIntern
4849
id: table.id.table_id as i32,
4950
name: table.name().into(),
5051
schema_id: schema.id() as i32,
52+
job_id: table.job_id.unwrap().table_id as i32,
5153
owner: table.owner as i32,
5254
definition: table.create_sql(),
5355
acl: get_acl_items(

src/frontend/src/catalog/system_catalog/rw_catalog/rw_streaming_parallelism.rs

Lines changed: 2 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -26,6 +26,8 @@ use risingwave_frontend_macro::system_catalog;
2626
SELECT id, name, 'sink' as relation_type FROM rw_sinks
2727
UNION ALL
2828
SELECT id, name, 'index' as relation_type FROM rw_indexes
29+
UNION ALL
30+
SELECT id, name, 'source' as relation_type FROM rw_sources WHERE is_shared = true
2931
)
3032
SELECT
3133
job.id,

src/frontend/src/catalog/table_catalog.rs

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -182,6 +182,8 @@ pub struct TableCatalog {
182182
pub vnode_count: VnodeCount,
183183

184184
pub webhook_info: Option<PbWebhookSourceInfo>,
185+
186+
pub job_id: Option<TableId>,
185187
}
186188

187189
#[derive(Copy, Clone, Debug, PartialEq, Eq, Hash)]
@@ -467,6 +469,7 @@ impl TableCatalog {
467469
cdc_table_id: self.cdc_table_id.clone(),
468470
maybe_vnode_count: self.vnode_count.to_protobuf(),
469471
webhook_info: self.webhook_info.clone(),
472+
job_id: self.job_id.map(|id| id.table_id),
470473
}
471474
}
472475

@@ -660,6 +663,7 @@ impl From<PbTable> for TableCatalog {
660663
cdc_table_id: tb.cdc_table_id,
661664
vnode_count,
662665
webhook_info: tb.webhook_info,
666+
job_id: tb.job_id.map(TableId::from),
663667
}
664668
}
665669
}
@@ -752,6 +756,7 @@ mod tests {
752756
cdc_table_id: None,
753757
maybe_vnode_count: VnodeCount::set(233).to_protobuf(),
754758
webhook_info: None,
759+
job_id: None,
755760
}
756761
.into();
757762

@@ -820,6 +825,7 @@ mod tests {
820825
cdc_table_id: None,
821826
vnode_count: VnodeCount::set(233),
822827
webhook_info: None,
828+
job_id: None,
823829
}
824830
);
825831
assert_eq!(table, TableCatalog::from(table.to_prost(0, 0)));

src/frontend/src/optimizer/plan_node/stream_materialize.rs

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -291,6 +291,7 @@ impl StreamMaterialize {
291291
cdc_table_id: None,
292292
vnode_count: VnodeCount::Placeholder, // will be filled in by the meta service later
293293
webhook_info,
294+
job_id: None,
294295
})
295296
}
296297

src/frontend/src/optimizer/plan_node/utils.rs

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -197,6 +197,7 @@ impl TableCatalogBuilder {
197197
cdc_table_id: None,
198198
vnode_count: VnodeCount::Placeholder, // will be filled in by the meta service later
199199
webhook_info: None,
200+
job_id: None,
200201
}
201202
}
202203

src/frontend/src/scheduler/distributed/query.rs

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -591,6 +591,7 @@ pub(crate) mod tests {
591591
cdc_table_id: None,
592592
vnode_count: VnodeCount::set(vnode_count),
593593
webhook_info: None,
594+
job_id: None,
594595
};
595596
let batch_plan_node: PlanRef = LogicalScan::create(
596597
"".to_string(),

0 commit comments

Comments
 (0)