risingwave_frontend/optimizer/plan_node/
batch_update.rs1use risingwave_common::catalog::Schema;
16use risingwave_pb::batch_plan::UpdateNode;
17use risingwave_pb::batch_plan::plan_node::NodeBody;
18
19use super::batch::prelude::*;
20use super::utils::impl_distill_by_unit;
21use super::{
22 BatchPlanRef as PlanRef, ExprRewritable, PlanBase, PlanTreeNodeUnary, ToBatchPb,
23 ToDistributedBatch, generic,
24};
25use crate::error::Result;
26use crate::expr::{Expr, ExprRewriter, ExprVisitor};
27use crate::optimizer::plan_node::expr_visitable::ExprVisitable;
28use crate::optimizer::plan_node::{ToLocalBatch, utils};
29use crate::optimizer::plan_visitor::DistributedDmlVisitor;
30use crate::optimizer::property::{Distribution, Order, RequiredDist};
31
32#[derive(Debug, Clone, PartialEq, Eq, Hash)]
34pub struct BatchUpdate {
35 pub base: PlanBase<Batch>,
36 pub core: generic::Update<PlanRef>,
37}
38
39impl BatchUpdate {
40 pub fn new(core: generic::Update<PlanRef>, schema: Schema) -> Self {
41 let ctx = core.input.ctx();
42 let base =
43 PlanBase::new_batch(ctx, schema, core.input.distribution().clone(), Order::any());
44 Self { base, core }
45 }
46}
47
48impl PlanTreeNodeUnary<Batch> for BatchUpdate {
49 fn input(&self) -> PlanRef {
50 self.core.input.clone()
51 }
52
53 fn clone_with_input(&self, input: PlanRef) -> Self {
54 let mut core = self.core.clone();
55 core.input = input;
56 Self::new(core, self.schema().clone())
57 }
58}
59
60impl_plan_tree_node_for_unary! { Batch, BatchUpdate }
61impl_distill_by_unit!(BatchUpdate, core, "BatchUpdate");
62
63impl ToDistributedBatch for BatchUpdate {
64 fn to_distributed(&self) -> Result<PlanRef> {
65 if DistributedDmlVisitor::dml_should_run_in_distributed(self.input()) {
66 let new_input = RequiredDist::PhysicalDist(Distribution::HashShard(
68 (0..self.input().schema().len()).collect(),
69 ))
70 .batch_enforce_if_not_satisfies(self.input().to_distributed()?, &Order::any())?;
71 let new_update = self.clone_with_input(new_input).into();
72 if self.core.returning {
73 Ok(new_update)
74 } else {
75 utils::sum_affected_row(new_update)
76 }
77 } else {
78 let new_input = RequiredDist::single()
79 .batch_enforce_if_not_satisfies(self.input().to_distributed()?, &Order::any())?;
80 Ok(self.clone_with_input(new_input).into())
81 }
82 }
83}
84
85impl ToBatchPb for BatchUpdate {
86 fn to_batch_prost_body(&self) -> NodeBody {
87 let old_exprs = (self.core.old_exprs)
88 .iter()
89 .map(|x| x.to_expr_proto())
90 .collect();
91 let new_exprs = (self.core.new_exprs)
92 .iter()
93 .map(|x| x.to_expr_proto())
94 .collect();
95
96 NodeBody::Update(UpdateNode {
97 table_id: self.core.table_id.table_id(),
98 table_version_id: self.core.table_version_id,
99 returning: self.core.returning,
100 old_exprs,
101 new_exprs,
102 session_id: self.base.ctx().session_ctx().session_id().0 as u32,
103 })
104 }
105}
106
107impl ToLocalBatch for BatchUpdate {
108 fn to_local(&self) -> Result<PlanRef> {
109 let new_input = RequiredDist::single()
110 .batch_enforce_if_not_satisfies(self.input().to_local()?, &Order::any())?;
111 Ok(self.clone_with_input(new_input).into())
112 }
113}
114
115impl ExprRewritable<Batch> for BatchUpdate {
116 fn has_rewritable_expr(&self) -> bool {
117 true
118 }
119
120 fn rewrite_exprs(&self, r: &mut dyn ExprRewriter) -> PlanRef {
121 let mut core = self.core.clone();
122 core.rewrite_exprs(r);
123 Self::new(core, self.schema().clone()).into()
124 }
125}
126
127impl ExprVisitable for BatchUpdate {
128 fn visit_exprs(&self, v: &mut dyn ExprVisitor) {
129 self.core.visit_exprs(v);
130 }
131}