datafusion_expr/
udwf.rs

1// Licensed to the Apache Software Foundation (ASF) under one
2// or more contributor license agreements.  See the NOTICE file
3// distributed with this work for additional information
4// regarding copyright ownership.  The ASF licenses this file
5// to you under the Apache License, Version 2.0 (the
6// "License"); you may not use this file except in compliance
7// with the License.  You may obtain a copy of the License at
8//
9//   http://www.apache.org/licenses/LICENSE-2.0
10//
11// Unless required by applicable law or agreed to in writing,
12// software distributed under the License is distributed on an
13// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
14// KIND, either express or implied.  See the License for the
15// specific language governing permissions and limitations
16// under the License.
17
18//! [`WindowUDF`]: User Defined Window Functions
19
20use arrow::compute::SortOptions;
21use std::cmp::Ordering;
22use std::hash::{Hash, Hasher};
23use std::{
24    any::Any,
25    fmt::{self, Debug, Display, Formatter},
26    sync::Arc,
27};
28
29use arrow::datatypes::{DataType, FieldRef};
30
31use crate::expr::WindowFunction;
32use crate::udf_eq::UdfEq;
33use crate::{
34    function::WindowFunctionSimplification, Expr, PartitionEvaluator, Signature,
35};
36use datafusion_common::{not_impl_err, Result};
37use datafusion_doc::Documentation;
38use datafusion_expr_common::dyn_eq::{DynEq, DynHash};
39use datafusion_functions_window_common::expr::ExpressionArgs;
40use datafusion_functions_window_common::field::WindowUDFFieldArgs;
41use datafusion_functions_window_common::partition::PartitionEvaluatorArgs;
42use datafusion_physical_expr_common::physical_expr::PhysicalExpr;
43
44/// Logical representation of a user-defined window function (UDWF).
45///
46/// A Window Function is called via the SQL `OVER` clause:
47///
48/// ```sql
49/// SELECT first_value(col) OVER (PARTITION BY a, b ORDER BY c) FROM foo;
50/// ```
51///
52/// A UDWF is different from a user defined function (UDF) in that it is
53/// stateful across batches.
54///
55/// See the documentation on [`PartitionEvaluator`] for more details
56///
57/// 1. For simple use cases, use [`create_udwf`] (examples in
58///    [`simple_udwf.rs`]).
59///
60/// 2. For advanced use cases, use [`WindowUDFImpl`] which provides full API
61///    access (examples in [`advanced_udwf.rs`]).
62///
63/// # API Note
64/// This is a separate struct from `WindowUDFImpl` to maintain backwards
65/// compatibility with the older API.
66///
67/// [`PartitionEvaluator`]: crate::PartitionEvaluator
68/// [`create_udwf`]: crate::expr_fn::create_udwf
69/// [`simple_udwf.rs`]: https://github.com/apache/datafusion/blob/main/datafusion-examples/examples/simple_udwf.rs
70/// [`advanced_udwf.rs`]: https://github.com/apache/datafusion/blob/main/datafusion-examples/examples/advanced_udwf.rs
71#[derive(Debug, Clone, PartialOrd)]
72pub struct WindowUDF {
73    inner: Arc<dyn WindowUDFImpl>,
74}
75
76/// Defines how the WindowUDF is shown to users
77impl Display for WindowUDF {
78    fn fmt(&self, f: &mut Formatter) -> fmt::Result {
79        write!(f, "{}", self.name())
80    }
81}
82
83impl PartialEq for WindowUDF {
84    fn eq(&self, other: &Self) -> bool {
85        self.inner.dyn_eq(other.inner.as_any())
86    }
87}
88
89impl Eq for WindowUDF {}
90
91impl Hash for WindowUDF {
92    fn hash<H: Hasher>(&self, state: &mut H) {
93        self.inner.dyn_hash(state)
94    }
95}
96
97impl WindowUDF {
98    /// Create a new `WindowUDF` from a `[WindowUDFImpl]` trait object
99    ///
100    /// Note this is the same as using the `From` impl (`WindowUDF::from`)
101    pub fn new_from_impl<F>(fun: F) -> WindowUDF
102    where
103        F: WindowUDFImpl + 'static,
104    {
105        Self::new_from_shared_impl(Arc::new(fun))
106    }
107
108    /// Create a new `WindowUDF` from a `[WindowUDFImpl]` trait object
109    pub fn new_from_shared_impl(fun: Arc<dyn WindowUDFImpl>) -> WindowUDF {
110        Self { inner: fun }
111    }
112
113    /// Return the underlying [`WindowUDFImpl`] trait object for this function
114    pub fn inner(&self) -> &Arc<dyn WindowUDFImpl> {
115        &self.inner
116    }
117
118    /// Adds additional names that can be used to invoke this function, in
119    /// addition to `name`
120    ///
121    /// If you implement [`WindowUDFImpl`] directly you should return aliases directly.
122    pub fn with_aliases(self, aliases: impl IntoIterator<Item = &'static str>) -> Self {
123        Self::new_from_impl(AliasedWindowUDFImpl::new(Arc::clone(&self.inner), aliases))
124    }
125
126    /// creates a [`Expr`] that calls the window function with default
127    /// values for `order_by`, `partition_by`, `window_frame`.
128    ///
129    /// See [`ExprFunctionExt`] for details on setting these values.
130    ///
131    /// This utility allows using a user defined window function without
132    /// requiring access to the registry, such as with the DataFrame API.
133    ///
134    /// [`ExprFunctionExt`]: crate::expr_fn::ExprFunctionExt
135    pub fn call(&self, args: Vec<Expr>) -> Expr {
136        let fun = crate::WindowFunctionDefinition::WindowUDF(Arc::new(self.clone()));
137
138        Expr::from(WindowFunction::new(fun, args))
139    }
140
141    /// Returns this function's name
142    ///
143    /// See [`WindowUDFImpl::name`] for more details.
144    pub fn name(&self) -> &str {
145        self.inner.name()
146    }
147
148    /// Returns the aliases for this function.
149    pub fn aliases(&self) -> &[String] {
150        self.inner.aliases()
151    }
152
153    /// Returns this function's signature (what input types are accepted)
154    ///
155    /// See [`WindowUDFImpl::signature`] for more details.
156    pub fn signature(&self) -> &Signature {
157        self.inner.signature()
158    }
159
160    /// Do the function rewrite
161    ///
162    /// See [`WindowUDFImpl::simplify`] for more details.
163    pub fn simplify(&self) -> Option<WindowFunctionSimplification> {
164        self.inner.simplify()
165    }
166
167    /// Expressions that are passed to the [`PartitionEvaluator`].
168    ///
169    /// See [`WindowUDFImpl::expressions`] for more details.
170    pub fn expressions(&self, expr_args: ExpressionArgs) -> Vec<Arc<dyn PhysicalExpr>> {
171        self.inner.expressions(expr_args)
172    }
173    /// Return a `PartitionEvaluator` for evaluating this window function
174    pub fn partition_evaluator_factory(
175        &self,
176        partition_evaluator_args: PartitionEvaluatorArgs,
177    ) -> Result<Box<dyn PartitionEvaluator>> {
178        self.inner.partition_evaluator(partition_evaluator_args)
179    }
180
181    /// Returns the field of the final result of evaluating this window function.
182    ///
183    /// See [`WindowUDFImpl::field`] for more details.
184    pub fn field(&self, field_args: WindowUDFFieldArgs) -> Result<FieldRef> {
185        self.inner.field(field_args)
186    }
187
188    /// Returns custom result ordering introduced by this window function
189    /// which is used to update ordering equivalences.
190    ///
191    /// See [`WindowUDFImpl::sort_options`] for more details.
192    pub fn sort_options(&self) -> Option<SortOptions> {
193        self.inner.sort_options()
194    }
195
196    /// See [`WindowUDFImpl::coerce_types`] for more details.
197    pub fn coerce_types(&self, arg_types: &[DataType]) -> Result<Vec<DataType>> {
198        self.inner.coerce_types(arg_types)
199    }
200
201    /// Returns the reversed user-defined window function when the
202    /// order of evaluation is reversed.
203    ///
204    /// See [`WindowUDFImpl::reverse_expr`] for more details.
205    pub fn reverse_expr(&self) -> ReversedUDWF {
206        self.inner.reverse_expr()
207    }
208
209    /// Returns the documentation for this Window UDF.
210    ///
211    /// Documentation can be accessed programmatically as well as
212    /// generating publicly facing documentation.
213    pub fn documentation(&self) -> Option<&Documentation> {
214        self.inner.documentation()
215    }
216}
217
218impl<F> From<F> for WindowUDF
219where
220    F: WindowUDFImpl + Send + Sync + 'static,
221{
222    fn from(fun: F) -> Self {
223        Self::new_from_impl(fun)
224    }
225}
226
227/// Trait for implementing [`WindowUDF`].
228///
229/// This trait exposes the full API for implementing user defined window functions and
230/// can be used to implement any function.
231///
232/// While the trait depends on [`DynEq`] and [`DynHash`] traits, these should not be
233/// implemented directly. Instead, implement [`Eq`] and [`Hash`] and leverage the
234/// blanket implementations of [`DynEq`] and [`DynHash`].
235///
236/// See [`advanced_udwf.rs`] for a full example with complete implementation and
237/// [`WindowUDF`] for other available options.
238///
239///
240/// [`advanced_udwf.rs`]: https://github.com/apache/datafusion/blob/main/datafusion-examples/examples/advanced_udwf.rs
241/// # Basic Example
242/// ```
243/// # use std::any::Any;
244/// # use std::sync::LazyLock;
245/// # use arrow::datatypes::{DataType, Field, FieldRef};
246/// # use datafusion_common::{DataFusionError, plan_err, Result};
247/// # use datafusion_expr::{col, Signature, Volatility, PartitionEvaluator, WindowFrame, ExprFunctionExt, Documentation, LimitEffect};
248/// # use datafusion_expr::{WindowUDFImpl, WindowUDF};
249/// # use datafusion_functions_window_common::field::WindowUDFFieldArgs;
250/// # use datafusion_functions_window_common::partition::PartitionEvaluatorArgs;
251/// # use datafusion_expr::window_doc_sections::DOC_SECTION_ANALYTICAL;
252/// # use datafusion_physical_expr_common::physical_expr;
253/// # use std::sync::Arc;
254///
255/// #[derive(Debug, Clone, PartialEq, Eq, Hash)]
256/// struct SmoothIt {
257///   signature: Signature,
258/// }
259///
260/// impl SmoothIt {
261///   fn new() -> Self {
262///     Self {
263///       signature: Signature::uniform(1, vec![DataType::Int32], Volatility::Immutable),
264///      }
265///   }
266/// }
267///
268/// static DOCUMENTATION: LazyLock<Documentation> = LazyLock::new(|| {
269///     Documentation::builder(DOC_SECTION_ANALYTICAL, "smooths the windows", "smooth_it(2)")
270///         .with_argument("arg1", "The int32 number to smooth by")
271///         .build()
272/// });
273///
274/// fn get_doc() -> &'static Documentation {
275///     &DOCUMENTATION
276/// }
277///
278/// /// Implement the WindowUDFImpl trait for SmoothIt
279/// impl WindowUDFImpl for SmoothIt {
280///    fn as_any(&self) -> &dyn Any { self }
281///    fn name(&self) -> &str { "smooth_it" }
282///    fn signature(&self) -> &Signature { &self.signature }
283///    // The actual implementation would smooth the window
284///    fn partition_evaluator(
285///        &self,
286///        _partition_evaluator_args: PartitionEvaluatorArgs,
287///    ) -> Result<Box<dyn PartitionEvaluator>> {
288///        unimplemented!()
289///    }
290///    fn field(&self, field_args: WindowUDFFieldArgs) -> Result<FieldRef> {
291///      if let Some(DataType::Int32) = field_args.get_input_field(0).map(|f| f.data_type().clone()) {
292///        Ok(Field::new(field_args.name(), DataType::Int32, false).into())
293///      } else {
294///        plan_err!("smooth_it only accepts Int32 arguments")
295///      }
296///    }
297///    fn documentation(&self) -> Option<&Documentation> {
298///      Some(get_doc())
299///    }
300///     fn limit_effect(&self, _args: &[Arc<dyn physical_expr::PhysicalExpr>]) -> LimitEffect {
301///         LimitEffect::Unknown
302///     }
303/// }
304///
305/// // Create a new WindowUDF from the implementation
306/// let smooth_it = WindowUDF::from(SmoothIt::new());
307///
308/// // Call the function `add_one(col)`
309/// // smooth_it(speed) OVER (PARTITION BY car ORDER BY time ASC)
310/// let expr = smooth_it.call(vec![col("speed")])
311///     .partition_by(vec![col("car")])
312///     .order_by(vec![col("time").sort(true, true)])
313///     .window_frame(WindowFrame::new(None))
314///     .build()
315///     .unwrap();
316/// ```
317pub trait WindowUDFImpl: Debug + DynEq + DynHash + Send + Sync {
318    /// Returns this object as an [`Any`] trait object
319    fn as_any(&self) -> &dyn Any;
320
321    /// Returns this function's name
322    fn name(&self) -> &str;
323
324    /// Returns any aliases (alternate names) for this function.
325    ///
326    /// Note: `aliases` should only include names other than [`Self::name`].
327    /// Defaults to `[]` (no aliases)
328    fn aliases(&self) -> &[String] {
329        &[]
330    }
331
332    /// Returns the function's [`Signature`] for information about what input
333    /// types are accepted and the function's Volatility.
334    fn signature(&self) -> &Signature;
335
336    /// Returns the expressions that are passed to the [`PartitionEvaluator`].
337    fn expressions(&self, expr_args: ExpressionArgs) -> Vec<Arc<dyn PhysicalExpr>> {
338        expr_args.input_exprs().into()
339    }
340
341    /// Invoke the function, returning the [`PartitionEvaluator`] instance
342    fn partition_evaluator(
343        &self,
344        partition_evaluator_args: PartitionEvaluatorArgs,
345    ) -> Result<Box<dyn PartitionEvaluator>>;
346
347    /// Optionally apply per-UDWF simplification / rewrite rules.
348    ///
349    /// This can be used to apply function specific simplification rules during
350    /// optimization. The default implementation does nothing.
351    ///
352    /// Note that DataFusion handles simplifying arguments and  "constant
353    /// folding" (replacing a function call with constant arguments such as
354    /// `my_add(1,2) --> 3` ). Thus, there is no need to implement such
355    /// optimizations manually for specific UDFs.
356    ///
357    /// Example:
358    /// [`advanced_udwf.rs`]: <https://github.com/apache/arrow-datafusion/blob/main/datafusion-examples/examples/advanced_udwf.rs>
359    ///
360    /// # Returns
361    /// [None] if simplify is not defined or,
362    ///
363    /// Or, a closure with two arguments:
364    /// * 'window_function': [crate::expr::WindowFunction] for which simplified has been invoked
365    /// * 'info': [crate::simplify::SimplifyInfo]
366    fn simplify(&self) -> Option<WindowFunctionSimplification> {
367        None
368    }
369
370    /// The [`FieldRef`] of the final result of evaluating this window function.
371    ///
372    /// Call `field_args.name()` to get the fully qualified name for defining
373    /// the [`FieldRef`]. For a complete example see the implementation in the
374    /// [Basic Example](WindowUDFImpl#basic-example) section.
375    fn field(&self, field_args: WindowUDFFieldArgs) -> Result<FieldRef>;
376
377    /// Allows the window UDF to define a custom result ordering.
378    ///
379    /// By default, a window UDF doesn't introduce an ordering.
380    /// But when specified by a window UDF this is used to update
381    /// ordering equivalences.
382    fn sort_options(&self) -> Option<SortOptions> {
383        None
384    }
385
386    /// Coerce arguments of a function call to types that the function can evaluate.
387    ///
388    /// This function is only called if [`WindowUDFImpl::signature`] returns [`crate::TypeSignature::UserDefined`]. Most
389    /// UDWFs should return one of the other variants of `TypeSignature` which handle common
390    /// cases
391    ///
392    /// See the [type coercion module](crate::type_coercion)
393    /// documentation for more details on type coercion
394    ///
395    /// For example, if your function requires a floating point arguments, but the user calls
396    /// it like `my_func(1::int)` (aka with `1` as an integer), coerce_types could return `[DataType::Float64]`
397    /// to ensure the argument was cast to `1::double`
398    ///
399    /// # Parameters
400    /// * `arg_types`: The argument types of the arguments  this function with
401    ///
402    /// # Return value
403    /// A Vec the same length as `arg_types`. DataFusion will `CAST` the function call
404    /// arguments to these specific types.
405    fn coerce_types(&self, _arg_types: &[DataType]) -> Result<Vec<DataType>> {
406        not_impl_err!("Function {} does not implement coerce_types", self.name())
407    }
408
409    /// Allows customizing the behavior of the user-defined window
410    /// function when it is evaluated in reverse order.
411    fn reverse_expr(&self) -> ReversedUDWF {
412        ReversedUDWF::NotSupported
413    }
414
415    /// Returns the documentation for this Window UDF.
416    ///
417    /// Documentation can be accessed programmatically as well as
418    /// generating publicly facing documentation.
419    fn documentation(&self) -> Option<&Documentation> {
420        None
421    }
422
423    /// If not causal, returns the effect this function will have on the window
424    fn limit_effect(&self, _args: &[Arc<dyn PhysicalExpr>]) -> LimitEffect {
425        LimitEffect::Unknown
426    }
427}
428
429/// the effect this function will have on the limit pushdown
430pub enum LimitEffect {
431    /// Does not affect the limit (i.e. this is causal)
432    None,
433    /// Either undeclared, or dynamic (only evaluatable at run time)
434    Unknown,
435    /// Grow the limit by N rows
436    Relative(usize),
437    /// Limit needs to be at least N rows
438    Absolute(usize),
439}
440
441pub enum ReversedUDWF {
442    /// The result of evaluating the user-defined window function
443    /// remains identical when reversed.
444    Identical,
445    /// A window function which does not support evaluating the result
446    /// in reverse order.
447    NotSupported,
448    /// Customize the user-defined window function for evaluating the
449    /// result in reverse order.
450    Reversed(Arc<WindowUDF>),
451}
452
453impl PartialEq for dyn WindowUDFImpl {
454    fn eq(&self, other: &Self) -> bool {
455        self.dyn_eq(other.as_any())
456    }
457}
458
459// TODO (https://github.com/apache/datafusion/issues/17064) PartialOrd is not consistent with PartialEq for `dyn WindowUDFImpl` and it should be
460impl PartialOrd for dyn WindowUDFImpl {
461    fn partial_cmp(&self, other: &Self) -> Option<Ordering> {
462        match self.name().partial_cmp(other.name()) {
463            Some(Ordering::Equal) => self.signature().partial_cmp(other.signature()),
464            cmp => cmp,
465        }
466    }
467}
468
469/// WindowUDF that adds an alias to the underlying function. It is better to
470/// implement [`WindowUDFImpl`], which supports aliases, directly if possible.
471#[derive(Debug, PartialEq, Eq, Hash)]
472struct AliasedWindowUDFImpl {
473    inner: UdfEq<Arc<dyn WindowUDFImpl>>,
474    aliases: Vec<String>,
475}
476
477impl AliasedWindowUDFImpl {
478    pub fn new(
479        inner: Arc<dyn WindowUDFImpl>,
480        new_aliases: impl IntoIterator<Item = &'static str>,
481    ) -> Self {
482        let mut aliases = inner.aliases().to_vec();
483        aliases.extend(new_aliases.into_iter().map(|s| s.to_string()));
484
485        Self {
486            inner: inner.into(),
487            aliases,
488        }
489    }
490}
491
492#[warn(clippy::missing_trait_methods)] // Delegates, so it should implement every single trait method
493impl WindowUDFImpl for AliasedWindowUDFImpl {
494    fn as_any(&self) -> &dyn Any {
495        self
496    }
497
498    fn name(&self) -> &str {
499        self.inner.name()
500    }
501
502    fn signature(&self) -> &Signature {
503        self.inner.signature()
504    }
505
506    fn expressions(&self, expr_args: ExpressionArgs) -> Vec<Arc<dyn PhysicalExpr>> {
507        expr_args
508            .input_exprs()
509            .first()
510            .map_or(vec![], |expr| vec![Arc::clone(expr)])
511    }
512
513    fn partition_evaluator(
514        &self,
515        partition_evaluator_args: PartitionEvaluatorArgs,
516    ) -> Result<Box<dyn PartitionEvaluator>> {
517        self.inner.partition_evaluator(partition_evaluator_args)
518    }
519
520    fn aliases(&self) -> &[String] {
521        &self.aliases
522    }
523
524    fn simplify(&self) -> Option<WindowFunctionSimplification> {
525        self.inner.simplify()
526    }
527
528    fn field(&self, field_args: WindowUDFFieldArgs) -> Result<FieldRef> {
529        self.inner.field(field_args)
530    }
531
532    fn sort_options(&self) -> Option<SortOptions> {
533        self.inner.sort_options()
534    }
535
536    fn coerce_types(&self, arg_types: &[DataType]) -> Result<Vec<DataType>> {
537        self.inner.coerce_types(arg_types)
538    }
539
540    fn reverse_expr(&self) -> ReversedUDWF {
541        self.inner.reverse_expr()
542    }
543
544    fn documentation(&self) -> Option<&Documentation> {
545        self.inner.documentation()
546    }
547
548    fn limit_effect(&self, args: &[Arc<dyn PhysicalExpr>]) -> LimitEffect {
549        self.inner.limit_effect(args)
550    }
551}
552
553// Window UDF doc sections for use in public documentation
554pub mod window_doc_sections {
555    use datafusion_doc::DocSection;
556
557    pub fn doc_sections() -> Vec<DocSection> {
558        vec![
559            DOC_SECTION_AGGREGATE,
560            DOC_SECTION_RANKING,
561            DOC_SECTION_ANALYTICAL,
562        ]
563    }
564
565    pub const DOC_SECTION_AGGREGATE: DocSection = DocSection {
566        include: true,
567        label: "Aggregate Functions",
568        description: Some("All aggregate functions can be used as window functions."),
569    };
570
571    pub const DOC_SECTION_RANKING: DocSection = DocSection {
572        include: true,
573        label: "Ranking Functions",
574        description: None,
575    };
576
577    pub const DOC_SECTION_ANALYTICAL: DocSection = DocSection {
578        include: true,
579        label: "Analytical Functions",
580        description: None,
581    };
582}
583
584#[cfg(test)]
585mod test {
586    use crate::{LimitEffect, PartitionEvaluator, WindowUDF, WindowUDFImpl};
587    use arrow::datatypes::{DataType, FieldRef};
588    use datafusion_common::Result;
589    use datafusion_expr_common::signature::{Signature, Volatility};
590    use datafusion_functions_window_common::field::WindowUDFFieldArgs;
591    use datafusion_functions_window_common::partition::PartitionEvaluatorArgs;
592    use datafusion_physical_expr_common::physical_expr::PhysicalExpr;
593    use std::any::Any;
594    use std::cmp::Ordering;
595    use std::hash::{DefaultHasher, Hash, Hasher};
596    use std::sync::Arc;
597
598    #[derive(Debug, Clone, PartialEq, Eq, Hash)]
599    struct AWindowUDF {
600        signature: Signature,
601    }
602
603    impl AWindowUDF {
604        fn new() -> Self {
605            Self {
606                signature: Signature::uniform(
607                    1,
608                    vec![DataType::Int32],
609                    Volatility::Immutable,
610                ),
611            }
612        }
613    }
614
615    /// Implement the WindowUDFImpl trait for AddOne
616    impl WindowUDFImpl for AWindowUDF {
617        fn as_any(&self) -> &dyn Any {
618            self
619        }
620        fn name(&self) -> &str {
621            "a"
622        }
623        fn signature(&self) -> &Signature {
624            &self.signature
625        }
626        fn partition_evaluator(
627            &self,
628            _partition_evaluator_args: PartitionEvaluatorArgs,
629        ) -> Result<Box<dyn PartitionEvaluator>> {
630            unimplemented!()
631        }
632        fn field(&self, _field_args: WindowUDFFieldArgs) -> Result<FieldRef> {
633            unimplemented!()
634        }
635
636        fn limit_effect(&self, _args: &[Arc<dyn PhysicalExpr>]) -> LimitEffect {
637            LimitEffect::Unknown
638        }
639    }
640
641    #[derive(Debug, Clone, PartialEq, Eq, Hash)]
642    struct BWindowUDF {
643        signature: Signature,
644    }
645
646    impl BWindowUDF {
647        fn new() -> Self {
648            Self {
649                signature: Signature::uniform(
650                    1,
651                    vec![DataType::Int32],
652                    Volatility::Immutable,
653                ),
654            }
655        }
656    }
657
658    /// Implement the WindowUDFImpl trait for AddOne
659    impl WindowUDFImpl for BWindowUDF {
660        fn as_any(&self) -> &dyn Any {
661            self
662        }
663        fn name(&self) -> &str {
664            "b"
665        }
666        fn signature(&self) -> &Signature {
667            &self.signature
668        }
669        fn partition_evaluator(
670            &self,
671            _partition_evaluator_args: PartitionEvaluatorArgs,
672        ) -> Result<Box<dyn PartitionEvaluator>> {
673            unimplemented!()
674        }
675        fn field(&self, _field_args: WindowUDFFieldArgs) -> Result<FieldRef> {
676            unimplemented!()
677        }
678
679        fn limit_effect(&self, _args: &[Arc<dyn PhysicalExpr>]) -> LimitEffect {
680            LimitEffect::Unknown
681        }
682    }
683
684    #[test]
685    fn test_partial_eq() {
686        let a1 = WindowUDF::from(AWindowUDF::new());
687        let a2 = WindowUDF::from(AWindowUDF::new());
688        let eq = a1 == a2;
689        assert!(eq);
690        assert_eq!(a1, a2);
691        assert_eq!(hash(a1), hash(a2));
692    }
693
694    #[test]
695    fn test_partial_ord() {
696        let a1 = WindowUDF::from(AWindowUDF::new());
697        let a2 = WindowUDF::from(AWindowUDF::new());
698        assert_eq!(a1.partial_cmp(&a2), Some(Ordering::Equal));
699
700        let b1 = WindowUDF::from(BWindowUDF::new());
701        assert!(a1 < b1);
702        assert!(!(a1 == b1));
703    }
704
705    fn hash<T: Hash>(value: T) -> u64 {
706        let hasher = &mut DefaultHasher::new();
707        value.hash(hasher);
708        hasher.finish()
709    }
710}