risingwave_frontend/rpc/
mod.rs1use itertools::Itertools;
16use pgwire::pg_server::{Session, SessionManager};
17use risingwave_common::id::{DatabaseId, TableId};
18use risingwave_pb::ddl_service::{ReplaceJobPlan, TableSchemaChange, replace_job_plan};
19use risingwave_pb::frontend_service::frontend_service_server::FrontendService;
20use risingwave_pb::frontend_service::{
21 CancelRunningSqlRequest, CancelRunningSqlResponse, GetRunningSqlsRequest,
22 GetRunningSqlsResponse, GetTableReplacePlanRequest, GetTableReplacePlanResponse, RunningSql,
23};
24use risingwave_sqlparser::ast::ObjectName;
25use tonic::{Request as RpcRequest, Response as RpcResponse, Status};
26
27use crate::error::RwError;
28use crate::handler::create_source::SqlColumnStrategy;
29use crate::handler::kill_process::handle_kill_local;
30use crate::handler::{get_new_table_definition_for_cdc_table, get_replace_table_plan};
31use crate::session::{SESSION_MANAGER, SessionMapRef};
32
33#[derive(Default)]
34pub struct FrontendServiceImpl {
35 session_map: SessionMapRef,
36}
37
38impl FrontendServiceImpl {
39 pub fn new(session_map: SessionMapRef) -> Self {
40 Self { session_map }
41 }
42}
43
44#[async_trait::async_trait]
45impl FrontendService for FrontendServiceImpl {
46 async fn get_table_replace_plan(
47 &self,
48 request: RpcRequest<GetTableReplacePlanRequest>,
49 ) -> Result<RpcResponse<GetTableReplacePlanResponse>, Status> {
50 let req = request.into_inner();
51
52 let replace_plan = get_new_table_plan(
53 req.table_id,
54 req.database_id,
55 req.owner,
56 req.cdc_table_change,
57 )
58 .await?;
59
60 Ok(RpcResponse::new(GetTableReplacePlanResponse {
61 replace_plan: Some(replace_plan),
62 }))
63 }
64
65 async fn get_running_sqls(
66 &self,
67 _request: RpcRequest<GetRunningSqlsRequest>,
68 ) -> Result<RpcResponse<GetRunningSqlsResponse>, Status> {
69 let running_sqls = self
70 .session_map
71 .read()
72 .values()
73 .map(|s| RunningSql {
74 process_id: s.id().0,
75 user_name: s.user_name(),
76 peer_addr: format!("{}", s.peer_addr()),
77 database: s.database(),
78 elapsed_millis: s.elapse_since_running_sql().and_then(|e| e.try_into().ok()),
79 sql: s.running_sql().map(|sql| format!("{}", sql)),
80 })
81 .collect();
82 Ok(RpcResponse::new(GetRunningSqlsResponse { running_sqls }))
83 }
84
85 async fn cancel_running_sql(
86 &self,
87 request: RpcRequest<CancelRunningSqlRequest>,
88 ) -> Result<RpcResponse<CancelRunningSqlResponse>, Status> {
89 let process_id = request.into_inner().process_id;
90 handle_kill_local(self.session_map.clone(), process_id).await?;
91 Ok(RpcResponse::new(CancelRunningSqlResponse {}))
92 }
93}
94
95async fn get_new_table_plan(
97 table_id: TableId,
98 database_id: DatabaseId,
99 owner: u32,
100 cdc_table_change: Option<TableSchemaChange>,
101) -> Result<ReplaceJobPlan, RwError> {
102 tracing::info!("get_new_table_plan for table {}", table_id);
103
104 let session_mgr = SESSION_MANAGER
105 .get()
106 .expect("session manager has been initialized");
107
108 let session = session_mgr.create_dummy_session(database_id, owner)?;
110
111 let table_catalog = {
112 let reader = session.env().catalog_reader().read_guard();
113 reader.get_any_table_by_id(table_id)?.clone()
114 };
115 let table_name = ObjectName::from(vec![table_catalog.name.as_str().into()]);
116
117 let definition = if let Some(cdc_table_change) = cdc_table_change {
118 let new_version_columns = cdc_table_change
119 .columns
120 .into_iter()
121 .map(|c| c.into())
122 .collect_vec();
123 get_new_table_definition_for_cdc_table(table_catalog.clone(), &new_version_columns).await?
124 } else {
125 table_catalog.create_sql_ast_purified()?
126 };
127
128 let (source, table, graph, job_type) = get_replace_table_plan(
129 &session,
130 table_name,
131 definition,
132 &table_catalog,
133 SqlColumnStrategy::FollowUnchecked,
134 )
135 .await?;
136
137 Ok(ReplaceJobPlan {
138 replace_job: Some(replace_job_plan::ReplaceJob::ReplaceTable(
139 replace_job_plan::ReplaceTable {
140 table: Some(table.to_prost()),
141 source: source.map(|s| s.to_prost()),
142 job_type: job_type as _,
143 },
144 )),
145 fragment_graph: Some(graph),
146 })
147}