mz_adapter/optimize/
subscribe.rs

1// Copyright Materialize, Inc. and contributors. All rights reserved.
2//
3// Use of this software is governed by the Business Source License
4// included in the LICENSE file.
5//
6// As of the Change Date specified in that file, in accordance with
7// the Business Source License, use of this software will be governed
8// by the Apache License, Version 2.0.
9
10//! Optimizer implementation for `SUBSCRIBE` statements.
11
12use std::marker::PhantomData;
13use std::sync::Arc;
14use std::time::{Duration, Instant};
15
16use differential_dataflow::lattice::Lattice;
17use mz_compute_types::ComputeInstanceId;
18use mz_compute_types::plan::Plan;
19use mz_compute_types::sinks::{ComputeSinkConnection, ComputeSinkDesc, SubscribeSinkConnection};
20use mz_ore::collections::CollectionExt;
21use mz_ore::soft_assert_or_log;
22use mz_repr::{GlobalId, RelationDesc, Timestamp};
23use mz_sql::optimizer_metrics::OptimizerMetrics;
24use mz_sql::plan::SubscribeFrom;
25use mz_transform::TransformCtx;
26use mz_transform::dataflow::DataflowMetainfo;
27use mz_transform::normalize_lets::normalize_lets;
28use mz_transform::reprtypecheck::{
29    SharedContext as ReprTypecheckContext, empty_context as empty_repr_context,
30};
31use timely::progress::Antichain;
32
33use crate::CollectionIdBundle;
34use crate::optimize::dataflows::{
35    ComputeInstanceSnapshot, DataflowBuilder, ExprPrepStyle, dataflow_import_id_bundle,
36    prep_relation_expr, prep_scalar_expr,
37};
38use crate::optimize::{
39    LirDataflowDescription, MirDataflowDescription, Optimize, OptimizeMode, OptimizerCatalog,
40    OptimizerConfig, OptimizerError, optimize_mir_local, trace_plan,
41};
42
43pub struct Optimizer {
44    /// A representation typechecking context to use throughout the optimizer pipeline.
45    repr_typecheck_ctx: ReprTypecheckContext,
46    /// A snapshot of the catalog state.
47    catalog: Arc<dyn OptimizerCatalog>,
48    /// A snapshot of the cluster that will run the dataflows.
49    compute_instance: ComputeInstanceSnapshot,
50    /// A transient GlobalId to be used for the exported sink.
51    sink_id: GlobalId,
52    /// A transient GlobalId to be used when constructing a dataflow for
53    /// `SUBSCRIBE FROM <SELECT>` variants.
54    view_id: GlobalId,
55    /// Should the plan produce an initial snapshot?
56    with_snapshot: bool,
57    /// Sink timestamp.
58    up_to: Option<Timestamp>,
59    /// A human-readable name exposed internally (useful for debugging).
60    debug_name: String,
61    /// Optimizer config.
62    config: OptimizerConfig,
63    /// Optimizer metrics.
64    metrics: OptimizerMetrics,
65    /// The time spent performing optimization so far.
66    duration: Duration,
67}
68
69// A bogey `Debug` implementation that hides fields. This is needed to make the
70// `event!` call in `sequence_peek_stage` not emit a lot of data.
71//
72// For now, we skip almost all fields, but we might revisit that bit if it turns
73// out that we really need those for debugging purposes.
74impl std::fmt::Debug for Optimizer {
75    fn fmt(&self, f: &mut std::fmt::Formatter<'_>) -> std::fmt::Result {
76        f.debug_struct("Optimizer")
77            .field("config", &self.config)
78            .finish_non_exhaustive()
79    }
80}
81
82impl Optimizer {
83    pub fn new(
84        catalog: Arc<dyn OptimizerCatalog>,
85        compute_instance: ComputeInstanceSnapshot,
86        view_id: GlobalId,
87        sink_id: GlobalId,
88        with_snapshot: bool,
89        up_to: Option<Timestamp>,
90        debug_name: String,
91        config: OptimizerConfig,
92        metrics: OptimizerMetrics,
93    ) -> Self {
94        Self {
95            repr_typecheck_ctx: empty_repr_context(),
96            catalog,
97            compute_instance,
98            view_id,
99            sink_id,
100            with_snapshot,
101            up_to,
102            debug_name,
103            config,
104            metrics,
105            duration: Default::default(),
106        }
107    }
108
109    pub fn cluster_id(&self) -> ComputeInstanceId {
110        self.compute_instance.instance_id()
111    }
112
113    pub fn up_to(&self) -> Option<Timestamp> {
114        self.up_to.clone()
115    }
116}
117
118/// The (sealed intermediate) result after:
119///
120/// 1. embedding a [`SubscribeFrom`] plan into a [`MirDataflowDescription`],
121/// 2. transitively inlining referenced views, and
122/// 3. jointly optimizing the `MIR` plans in the [`MirDataflowDescription`].
123#[derive(Clone, Debug)]
124pub struct GlobalMirPlan<T: Clone> {
125    df_desc: MirDataflowDescription,
126    df_meta: DataflowMetainfo,
127    phantom: PhantomData<T>,
128}
129
130impl<T: Clone> GlobalMirPlan<T> {
131    /// Computes the [`CollectionIdBundle`] of the wrapped dataflow.
132    pub fn id_bundle(&self, compute_instance_id: ComputeInstanceId) -> CollectionIdBundle {
133        dataflow_import_id_bundle(&self.df_desc, compute_instance_id)
134    }
135}
136
137/// The (final) result after MIR ⇒ LIR lowering and optimizing the resulting
138/// `DataflowDescription` with `LIR` plans.
139#[derive(Clone, Debug)]
140pub struct GlobalLirPlan {
141    df_desc: LirDataflowDescription,
142    df_meta: DataflowMetainfo,
143}
144
145impl GlobalLirPlan {
146    /// Returns the id of the dataflow's sink export.
147    ///
148    /// # Panics
149    ///
150    /// Panics if the dataflow has no sink exports or has more than one.
151    pub fn sink_id(&self) -> GlobalId {
152        self.df_desc.sink_id()
153    }
154
155    pub fn as_of(&self) -> Option<Timestamp> {
156        self.df_desc.as_of.clone().map(|as_of| as_of.into_element())
157    }
158
159    /// Returns the description of the dataflow's sink export.
160    ///
161    /// # Panics
162    ///
163    /// Panics if the dataflow has no sink exports or has more than one.
164    pub fn sink_desc(&self) -> &ComputeSinkDesc {
165        let sink_exports = &self.df_desc.sink_exports;
166        let sink_desc = sink_exports.values().into_element();
167        sink_desc
168    }
169}
170
171/// Marker type for [`GlobalMirPlan`] structs representing an optimization
172/// result without a resolved timestamp.
173#[derive(Clone, Debug)]
174pub struct Unresolved;
175
176/// Marker type for [`GlobalMirPlan`] structs representing an optimization
177/// result with a resolved timestamp.
178///
179/// The actual timestamp value is set in the [`MirDataflowDescription`] of the
180/// surrounding [`GlobalMirPlan`] when we call `resolve()`.
181#[derive(Clone, Debug)]
182pub struct Resolved;
183
184impl Optimize<SubscribeFrom> for Optimizer {
185    type To = GlobalMirPlan<Unresolved>;
186
187    fn optimize(&mut self, plan: SubscribeFrom) -> Result<Self::To, OptimizerError> {
188        let time = Instant::now();
189
190        let mut df_builder = {
191            let compute = self.compute_instance.clone();
192            DataflowBuilder::new(&*self.catalog, compute).with_config(&self.config)
193        };
194        let mut df_desc = MirDataflowDescription::new(self.debug_name.clone());
195        let mut df_meta = DataflowMetainfo::default();
196
197        match plan {
198            SubscribeFrom::Id(from_id) => {
199                let from = self.catalog.get_entry(&from_id);
200                let from_desc = from
201                    .relation_desc()
202                    .expect("subscribes can only be run on items with descs")
203                    .into_owned();
204
205                df_builder.import_into_dataflow(&from_id, &mut df_desc, &self.config.features)?;
206                df_builder.maybe_reoptimize_imported_views(&mut df_desc, &self.config)?;
207
208                // Make SinkDesc
209                let sink_description = ComputeSinkDesc {
210                    from: from_id,
211                    from_desc,
212                    connection: ComputeSinkConnection::Subscribe(SubscribeSinkConnection::default()),
213                    with_snapshot: self.with_snapshot,
214                    up_to: self.up_to.map(Antichain::from_elem).unwrap_or_default(),
215                    // No `FORCE NOT NULL` for subscribes
216                    non_null_assertions: vec![],
217                    // No `REFRESH` for subscribes
218                    refresh_schedule: None,
219                };
220                df_desc.export_sink(self.sink_id, sink_description);
221            }
222            SubscribeFrom::Query { expr, desc } => {
223                // TODO: Change the `expr` type to be `HirRelationExpr` and run
224                // HIR ⇒ MIR lowering and decorrelation here. This would allow
225                // us implement something like `EXPLAIN RAW PLAN FOR SUBSCRIBE.`
226                //
227                // let expr = expr.lower(&self.config)?;
228
229                // MIR ⇒ MIR optimization (local)
230                let mut transform_ctx = TransformCtx::local(
231                    &self.config.features,
232                    &self.repr_typecheck_ctx,
233                    &mut df_meta,
234                    Some(&mut self.metrics),
235                    Some(self.view_id),
236                );
237                let expr = optimize_mir_local(expr, &mut transform_ctx)?;
238
239                df_builder.import_view_into_dataflow(
240                    &self.view_id,
241                    &expr,
242                    &mut df_desc,
243                    &self.config.features,
244                )?;
245                df_builder.maybe_reoptimize_imported_views(&mut df_desc, &self.config)?;
246
247                // Make SinkDesc
248                let sink_description = ComputeSinkDesc {
249                    from: self.view_id,
250                    from_desc: RelationDesc::new(expr.typ(), desc.iter_names()),
251                    connection: ComputeSinkConnection::Subscribe(SubscribeSinkConnection::default()),
252                    with_snapshot: self.with_snapshot,
253                    up_to: self.up_to.map(Antichain::from_elem).unwrap_or_default(),
254                    // No `FORCE NOT NULL` for subscribes
255                    non_null_assertions: vec![],
256                    // No `REFRESH` for subscribes
257                    refresh_schedule: None,
258                };
259                df_desc.export_sink(self.sink_id, sink_description);
260            }
261        };
262
263        // Prepare expressions in the assembled dataflow.
264        let style = ExprPrepStyle::Maintained;
265        df_desc.visit_children(
266            |r| prep_relation_expr(r, style),
267            |s| prep_scalar_expr(s, style),
268        )?;
269
270        // Construct TransformCtx for global optimization.
271        let mut transform_ctx = TransformCtx::global(
272            &df_builder,
273            &mz_transform::EmptyStatisticsOracle, // TODO: wire proper stats
274            &self.config.features,
275            &self.repr_typecheck_ctx,
276            &mut df_meta,
277            Some(&mut self.metrics),
278        );
279        // Run global optimization.
280        mz_transform::optimize_dataflow(&mut df_desc, &mut transform_ctx, false)?;
281
282        if self.config.mode == OptimizeMode::Explain {
283            // Collect the list of indexes used by the dataflow at this point.
284            trace_plan!(at: "global", &df_meta.used_indexes(&df_desc));
285        }
286
287        self.duration += time.elapsed();
288
289        // Return the (sealed) plan at the end of this optimization step.
290        Ok(GlobalMirPlan {
291            df_desc,
292            df_meta,
293            phantom: PhantomData::<Unresolved>,
294        })
295    }
296}
297
298impl GlobalMirPlan<Unresolved> {
299    /// Produces the [`GlobalMirPlan`] with [`Resolved`] timestamp.
300    ///
301    /// We need to resolve timestamps before the `GlobalMirPlan ⇒ GlobalLirPlan`
302    /// optimization stage in order to profit from possible single-time
303    /// optimizations in the `Plan::finalize_dataflow` call.
304    pub fn resolve(mut self, as_of: Antichain<Timestamp>) -> GlobalMirPlan<Resolved> {
305        // A dataflow description for a `SUBSCRIBE` statement should not have
306        // index exports.
307        soft_assert_or_log!(
308            self.df_desc.index_exports.is_empty(),
309            "unexpectedly setting until for a DataflowDescription with an index",
310        );
311
312        // Set the `as_of` timestamp for the dataflow.
313        self.df_desc.set_as_of(as_of);
314
315        // The only outputs of the dataflow are sinks, so we might be able to
316        // turn off the computation early, if they all have non-trivial
317        // `up_to`s.
318        self.df_desc.until = Antichain::from_elem(Timestamp::MIN);
319        for (_, sink) in &self.df_desc.sink_exports {
320            self.df_desc.until.join_assign(&sink.up_to);
321        }
322
323        GlobalMirPlan {
324            df_desc: self.df_desc,
325            df_meta: self.df_meta,
326            phantom: PhantomData::<Resolved>,
327        }
328    }
329}
330
331impl Optimize<GlobalMirPlan<Resolved>> for Optimizer {
332    type To = GlobalLirPlan;
333
334    fn optimize(&mut self, plan: GlobalMirPlan<Resolved>) -> Result<Self::To, OptimizerError> {
335        let time = Instant::now();
336
337        let GlobalMirPlan {
338            mut df_desc,
339            df_meta,
340            phantom: _,
341        } = plan;
342
343        // Ensure all expressions are normalized before finalizing.
344        for build in df_desc.objects_to_build.iter_mut() {
345            normalize_lets(&mut build.plan.0, &self.config.features)?
346        }
347
348        // Finalize the dataflow. This includes:
349        // - MIR ⇒ LIR lowering
350        // - LIR ⇒ LIR transforms
351        let df_desc = Plan::finalize_dataflow(df_desc, &self.config.features)?;
352
353        self.duration += time.elapsed();
354        self.metrics
355            .observe_e2e_optimization_time("subscribe", self.duration);
356
357        // Return the plan at the end of this `optimize` step.
358        Ok(GlobalLirPlan { df_desc, df_meta })
359    }
360}
361
362impl GlobalLirPlan {
363    /// Unwraps the parts of the final result of the optimization pipeline.
364    pub fn unapply(self) -> (LirDataflowDescription, DataflowMetainfo) {
365        (self.df_desc, self.df_meta)
366    }
367}