You are viewing a plain text version of this content. The canonical link for it is here.
Posted to github@arrow.apache.org by GitBox <gi...@apache.org> on 2021/06/15 06:02:39 UTC

[GitHub] [arrow-datafusion] Jimexist opened a new pull request #564: Add benchmarks to window function queries

Jimexist opened a new pull request #564:
URL: https://github.com/apache/arrow-datafusion/pull/564


   # Which issue does this PR close?
   
   Closes #.
   
    # Rationale for this change
   <!--
    Why are you proposing this change? If this is already explained clearly in the issue then this section is not needed.
    Explaining clearly why changes are proposed helps reviewers understand your changes and offer better suggestions for fixes.  
   -->
   
   # What changes are included in this PR?
   <!--
   There is no need to duplicate the description in the issue here but it is sometimes worth providing a summary of the individual changes in this PR.
   -->
   
   # Are there any user-facing changes?
   <!--
   If there are user-facing changes then we may require documentation to be updated before approving the PR.
   -->
   
   <!--
   If there are any breaking changes to public APIs, please add the `api change` label.
   -->
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [arrow-datafusion] Jimexist commented on pull request #564: Add benchmarks to window function queries

Posted by GitBox <gi...@apache.org>.
Jimexist commented on pull request #564:
URL: https://github.com/apache/arrow-datafusion/pull/564#issuecomment-865041167


   updated to 1million samples, 8k batch size:
   
   ```
   window empty over, aggregate functions
                           time:   [32.030 ms 32.251 ms 32.498 ms]
   Found 9 outliers among 100 measurements (9.00%)
     1 (1.00%) low mild
     4 (4.00%) high mild
     4 (4.00%) high severe
   
   window empty over, built-in functions
                           time:   [25.202 ms 25.453 ms 25.716 ms]
   Found 10 outliers among 100 measurements (10.00%)
     1 (1.00%) low mild
     5 (5.00%) high mild
     4 (4.00%) high severe
   
   Benchmarking window order by, aggregate functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 17.1s, or reduce sample count to 20.
   window order by, aggregate functions
                           time:   [168.14 ms 169.41 ms 170.74 ms]
   Found 7 outliers among 100 measurements (7.00%)
     3 (3.00%) low mild
     3 (3.00%) high mild
     1 (1.00%) high severe
   
   Benchmarking window order by, built-in functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 15.5s, or reduce sample count to 30.
   window order by, built-in functions
                           time:   [153.69 ms 155.38 ms 157.13 ms]
   Found 3 outliers among 100 measurements (3.00%)
     3 (3.00%) high mild
   
   Benchmarking window partition by, u64_wide, aggregate functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 25.5s, or reduce sample count to 10.
   window partition by, u64_wide, aggregate functions
                           time:   [249.29 ms 251.94 ms 254.74 ms]
   Found 1 outliers among 100 measurements (1.00%)
     1 (1.00%) high mild
   
   Benchmarking window partition by, u64_narrow, aggregate functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 12.8s, or reduce sample count to 30.
   window partition by, u64_narrow, aggregate functions
                           time:   [136.66 ms 138.32 ms 140.00 ms]
   Found 2 outliers among 100 measurements (2.00%)
     1 (1.00%) low mild
     1 (1.00%) high mild
   
   Benchmarking window partition by, u64_wide, built-in functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 23.0s, or reduce sample count to 20.
   window partition by, u64_wide, built-in functions
                           time:   [212.94 ms 214.92 ms 217.02 ms]
   Found 4 outliers among 100 measurements (4.00%)
     3 (3.00%) high mild
     1 (1.00%) high severe
   
   Benchmarking window partition by, u64_narrow, built-in functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 12.0s, or reduce sample count to 40.
   window partition by, u64_narrow, built-in functions
                           time:   [114.67 ms 115.70 ms 116.78 ms]
   Found 2 outliers among 100 measurements (2.00%)
     2 (2.00%) high mild
   
   Benchmarking window partition and order by, u64_wide, aggregate functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 42.3s, or reduce sample count to 10.
   window partition and order by, u64_wide, aggregate functions
                           time:   [424.65 ms 427.99 ms 431.40 ms]
   Found 1 outliers among 100 measurements (1.00%)
     1 (1.00%) high mild
   
   Benchmarking window partition and order by, u64_narrow, aggregate functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 62.7s, or reduce sample count to 10.
   window partition and order by, u64_narrow, aggregate functions
                           time:   [603.14 ms 607.66 ms 612.26 ms]
   
   Benchmarking window partition and order by, u64_wide, built-in functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 38.5s, or reduce sample count to 10.
   window partition and order by, u64_wide, built-in functions
                           time:   [381.45 ms 384.54 ms 387.73 ms]
   Found 2 outliers among 100 measurements (2.00%)
     2 (2.00%) high mild
   
   Benchmarking window partition and order by, u64_narrow, built-in functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 39.2s, or reduce sample count to 10.
   window partition and order by, u64_narrow, built-in functions
                           time:   [382.72 ms 385.97 ms 389.31 ms]
   Found 2 outliers among 100 measurements (2.00%)
     2 (2.00%) high mild
   
   ```


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [arrow-datafusion] Jimexist commented on a change in pull request #564: Add benchmarks to window function queries

Posted by GitBox <gi...@apache.org>.
Jimexist commented on a change in pull request #564:
URL: https://github.com/apache/arrow-datafusion/pull/564#discussion_r654078954



##########
File path: datafusion/benches/data_utils/mod.rs
##########
@@ -0,0 +1,155 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+//! This module provides the in-memory table for more realistic benchmarking.
+
+use arrow::{
+    array::Float32Array,
+    array::Float64Array,
+    array::StringArray,
+    array::UInt64Array,
+    datatypes::{DataType, Field, Schema, SchemaRef},
+    record_batch::RecordBatch,
+};
+use datafusion::datasource::MemTable;
+use datafusion::error::Result;
+use rand::rngs::StdRng;
+use rand::seq::SliceRandom;
+use rand::{Rng, SeedableRng};
+use std::sync::Arc;
+
+/// create an in-memory table given the partition len, array len, and batch size,
+/// and the result table will be of array_len in total, and then partitioned, and batched.
+pub(crate) fn create_table_provider(
+    partitions_len: usize,
+    array_len: usize,
+    batch_size: usize,
+) -> Result<Arc<MemTable>> {
+    let schema = Arc::new(create_schema());
+    let partitions =
+        create_record_batches(schema.clone(), array_len, partitions_len, batch_size);
+    // declare a table in memory. In spark API, this corresponds to createDataFrame(...).
+    MemTable::try_new(schema, partitions).map(Arc::new)
+}
+
+/// create a seedable [`StdRng`](rand::StdRng)
+fn seedable_rng() -> StdRng {
+    StdRng::seed_from_u64(42)
+}
+
+fn create_schema() -> Schema {
+    Schema::new(vec![
+        Field::new("utf8", DataType::Utf8, false),
+        Field::new("f32", DataType::Float32, false),
+        Field::new("f64", DataType::Float64, false),
+        // This field will contain integers randomly selected from a large
+        // range of values, i.e. [0, u64::MAX], such that there are none (or
+        // very few) repeated values.
+        Field::new("u64_wide", DataType::UInt64, false),
+        // This field will contain integers randomly selected from a narrow
+        // range of values such that there are a few distinct values, but they
+        // are repeated often.
+        Field::new("u64_narrow", DataType::UInt64, false),
+    ])
+}
+
+fn create_data(size: usize, null_density: f64) -> Vec<Option<f64>> {
+    // use random numbers to avoid spurious compiler optimizations wrt to branching
+    let mut rng = seedable_rng();
+
+    (0..size)
+        .map(|_| {
+            if rng.gen::<f64>() > null_density {
+                None
+            } else {
+                Some(rng.gen::<f64>())
+            }
+        })
+        .collect()
+}
+
+fn create_integer_data(size: usize, value_density: f64) -> Vec<Option<u64>> {
+    // use random numbers to avoid spurious compiler optimizations wrt to branching
+    let mut rng = seedable_rng();
+
+    (0..size)
+        .map(|_| {
+            if rng.gen::<f64>() > value_density {
+                None
+            } else {
+                Some(rng.gen::<u64>())
+            }
+        })
+        .collect()
+}
+
+fn create_record_batch(
+    schema: SchemaRef,
+    rng: &mut StdRng,
+    batch_size: usize,
+    i: usize,
+) -> RecordBatch {
+    // the 4 here is the number of different keys.
+    // a higher number increase sparseness
+    let vs = vec![0, 1, 2, 3];
+    let keys: Vec<String> = (0..batch_size)
+        .map(
+            // use random numbers to avoid spurious compiler optimizations wrt to branching
+            |_| format!("hi{:?}", vs.choose(rng)),
+        )
+        .collect();
+    let keys: Vec<&str> = keys.iter().map(|e| &**e).collect();
+
+    let values = create_data(batch_size, 0.5);
+
+    // Integer values between [0, u64::MAX].
+    let integer_values_wide = create_integer_data(batch_size, 9.0);
+
+    // Integer values between [0, 9].
+    let integer_values_narrow_choices = (0..10).collect::<Vec<u64>>();
+    let integer_values_narrow = (0..batch_size)
+        .map(|_| *integer_values_narrow_choices.choose(rng).unwrap())

Review comment:
       moved from another file so maybe i'll leave it as is.

##########
File path: datafusion/benches/data_utils/mod.rs
##########
@@ -0,0 +1,155 @@
+// Licensed to the Apache Software Foundation (ASF) under one
+// or more contributor license agreements.  See the NOTICE file
+// distributed with this work for additional information
+// regarding copyright ownership.  The ASF licenses this file
+// to you under the Apache License, Version 2.0 (the
+// "License"); you may not use this file except in compliance
+// with the License.  You may obtain a copy of the License at
+//
+//   http://www.apache.org/licenses/LICENSE-2.0
+//
+// Unless required by applicable law or agreed to in writing,
+// software distributed under the License is distributed on an
+// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+// KIND, either express or implied.  See the License for the
+// specific language governing permissions and limitations
+// under the License.
+
+//! This module provides the in-memory table for more realistic benchmarking.
+
+use arrow::{
+    array::Float32Array,
+    array::Float64Array,
+    array::StringArray,
+    array::UInt64Array,
+    datatypes::{DataType, Field, Schema, SchemaRef},
+    record_batch::RecordBatch,
+};
+use datafusion::datasource::MemTable;
+use datafusion::error::Result;
+use rand::rngs::StdRng;
+use rand::seq::SliceRandom;
+use rand::{Rng, SeedableRng};
+use std::sync::Arc;
+
+/// create an in-memory table given the partition len, array len, and batch size,
+/// and the result table will be of array_len in total, and then partitioned, and batched.
+pub(crate) fn create_table_provider(
+    partitions_len: usize,
+    array_len: usize,
+    batch_size: usize,
+) -> Result<Arc<MemTable>> {
+    let schema = Arc::new(create_schema());
+    let partitions =
+        create_record_batches(schema.clone(), array_len, partitions_len, batch_size);
+    // declare a table in memory. In spark API, this corresponds to createDataFrame(...).
+    MemTable::try_new(schema, partitions).map(Arc::new)
+}
+
+/// create a seedable [`StdRng`](rand::StdRng)
+fn seedable_rng() -> StdRng {
+    StdRng::seed_from_u64(42)
+}
+
+fn create_schema() -> Schema {
+    Schema::new(vec![
+        Field::new("utf8", DataType::Utf8, false),
+        Field::new("f32", DataType::Float32, false),
+        Field::new("f64", DataType::Float64, false),
+        // This field will contain integers randomly selected from a large
+        // range of values, i.e. [0, u64::MAX], such that there are none (or
+        // very few) repeated values.
+        Field::new("u64_wide", DataType::UInt64, false),
+        // This field will contain integers randomly selected from a narrow
+        // range of values such that there are a few distinct values, but they
+        // are repeated often.
+        Field::new("u64_narrow", DataType::UInt64, false),
+    ])
+}
+
+fn create_data(size: usize, null_density: f64) -> Vec<Option<f64>> {
+    // use random numbers to avoid spurious compiler optimizations wrt to branching
+    let mut rng = seedable_rng();
+
+    (0..size)
+        .map(|_| {
+            if rng.gen::<f64>() > null_density {
+                None
+            } else {
+                Some(rng.gen::<f64>())
+            }
+        })
+        .collect()
+}
+
+fn create_integer_data(size: usize, value_density: f64) -> Vec<Option<u64>> {
+    // use random numbers to avoid spurious compiler optimizations wrt to branching
+    let mut rng = seedable_rng();
+
+    (0..size)
+        .map(|_| {
+            if rng.gen::<f64>() > value_density {
+                None
+            } else {
+                Some(rng.gen::<u64>())
+            }
+        })
+        .collect()
+}
+
+fn create_record_batch(
+    schema: SchemaRef,
+    rng: &mut StdRng,
+    batch_size: usize,
+    i: usize,
+) -> RecordBatch {
+    // the 4 here is the number of different keys.
+    // a higher number increase sparseness
+    let vs = vec![0, 1, 2, 3];
+    let keys: Vec<String> = (0..batch_size)
+        .map(
+            // use random numbers to avoid spurious compiler optimizations wrt to branching
+            |_| format!("hi{:?}", vs.choose(rng)),
+        )
+        .collect();
+    let keys: Vec<&str> = keys.iter().map(|e| &**e).collect();
+
+    let values = create_data(batch_size, 0.5);
+
+    // Integer values between [0, u64::MAX].
+    let integer_values_wide = create_integer_data(batch_size, 9.0);
+
+    // Integer values between [0, 9].
+    let integer_values_narrow_choices = (0..10).collect::<Vec<u64>>();
+    let integer_values_narrow = (0..batch_size)
+        .map(|_| *integer_values_narrow_choices.choose(rng).unwrap())

Review comment:
       can change later though




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [arrow-datafusion] Dandandan merged pull request #564: Add benchmarks to window function queries

Posted by GitBox <gi...@apache.org>.
Dandandan merged pull request #564:
URL: https://github.com/apache/arrow-datafusion/pull/564


   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [arrow-datafusion] Jimexist commented on pull request #564: Add benchmarks to window function queries

Posted by GitBox <gi...@apache.org>.
Jimexist commented on pull request #564:
URL: https://github.com/apache/arrow-datafusion/pull/564#issuecomment-865005319


   @Dandandan @alamb this benchmark is now updated and rebased


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [arrow-datafusion] Jimexist commented on pull request #564: Add benchmarks to window function queries

Posted by GitBox <gi...@apache.org>.
Jimexist commented on pull request #564:
URL: https://github.com/apache/arrow-datafusion/pull/564#issuecomment-865005156


   updated benchmark:
   
   ```
   Benchmarking window empty over, aggregate functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 9.5s, enable flat sampling, or reduce sample count to 50.
   window empty over, aggregate functions
                           time:   [1.8108 ms 1.8229 ms 1.8366 ms]
   Found 9 outliers among 100 measurements (9.00%)
     6 (6.00%) high mild
     3 (3.00%) high severe
   
   Benchmarking window empty over, built-in functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 7.1s, enable flat sampling, or reduce sample count to 50.
   window empty over, built-in functions
                           time:   [1.3960 ms 1.4046 ms 1.4148 ms]
   Found 11 outliers among 100 measurements (11.00%)
     6 (6.00%) high mild
     5 (5.00%) high severe
   
   window order by, aggregate functions
                           time:   [8.4081 ms 8.5143 ms 8.6268 ms]
   Found 2 outliers among 100 measurements (2.00%)
     2 (2.00%) high mild
   
   window order by, built-in functions
                           time:   [7.4327 ms 7.4752 ms 7.5203 ms]
   Found 2 outliers among 100 measurements (2.00%)
     2 (2.00%) high mild
   
   window partition by, u64_wide, aggregate functions
                           time:   [17.877 ms 17.982 ms 18.094 ms]
   Found 2 outliers among 100 measurements (2.00%)
     2 (2.00%) high mild
   
   window partition by, u64_narrow, aggregate functions
                           time:   [6.4812 ms 6.5318 ms 6.5843 ms]
   Found 4 outliers among 100 measurements (4.00%)
     4 (4.00%) high mild
   
   window partition by, u64_wide, built-in functions
                           time:   [16.144 ms 16.248 ms 16.360 ms]
   Found 9 outliers among 100 measurements (9.00%)
     6 (6.00%) high mild
     3 (3.00%) high severe
   
   window partition by, u64_narrow, built-in functions
                           time:   [5.8226 ms 5.8595 ms 5.8988 ms]
   Found 11 outliers among 100 measurements (11.00%)
     11 (11.00%) high mild
   
   window partition and order by, u64_wide, aggregate functions
                           time:   [25.350 ms 25.578 ms 25.824 ms]
   Found 4 outliers among 100 measurements (4.00%)
     4 (4.00%) high mild
   
   Benchmarking window partition and order by, u64_narrow, aggregate functions: Warming up for 3.0000 s
   Warning: Unable to complete 100 samples in 5.0s. You may wish to increase target time to 5.4s, or reduce sample count to 90.
   window partition and order by, u64_narrow, aggregate functions
                           time:   [53.310 ms 53.726 ms 54.169 ms]
   Found 3 outliers among 100 measurements (3.00%)
     3 (3.00%) high mild
   
   window partition and order by, u64_wide, built-in functions
                           time:   [23.481 ms 23.690 ms 23.917 ms]
   Found 3 outliers among 100 measurements (3.00%)
     2 (2.00%) high mild
     1 (1.00%) high severe
   
   window partition and order by, u64_narrow, built-in functions
                           time:   [15.603 ms 15.715 ms 15.839 ms]
   Found 6 outliers among 100 measurements (6.00%)
     2 (2.00%) high mild
     4 (4.00%) high severe
   ```


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [arrow-datafusion] codecov-commenter commented on pull request #564: Add benchmarks to window function queries

Posted by GitBox <gi...@apache.org>.
codecov-commenter commented on pull request #564:
URL: https://github.com/apache/arrow-datafusion/pull/564#issuecomment-861212326


   # [Codecov](https://codecov.io/gh/apache/arrow-datafusion/pull/564?src=pr&el=h1&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) Report
   > Merging [#564](https://codecov.io/gh/apache/arrow-datafusion/pull/564?src=pr&el=desc&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) (573d7b5) into [master](https://codecov.io/gh/apache/arrow-datafusion/commit/e3e7e293c4482af1475406bf4e922d5c99e7a792?el=desc&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) (e3e7e29) will **decrease** coverage by `0.03%`.
   > The diff coverage is `81.01%`.
   
   [![Impacted file tree graph](https://codecov.io/gh/apache/arrow-datafusion/pull/564/graphs/tree.svg?width=650&height=150&src=pr&token=JXwWBKD3D9&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation)](https://codecov.io/gh/apache/arrow-datafusion/pull/564?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation)
   
   ```diff
   @@            Coverage Diff             @@
   ##           master     #564      +/-   ##
   ==========================================
   - Coverage   76.12%   76.08%   -0.04%     
   ==========================================
     Files         156      156              
     Lines       27074    27121      +47     
   ==========================================
   + Hits        20609    20635      +26     
   - Misses       6465     6486      +21     
   ```
   
   
   | [Impacted Files](https://codecov.io/gh/apache/arrow-datafusion/pull/564?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) | Coverage Δ | |
   |---|---|---|
   | [datafusion/src/physical\_plan/window\_functions.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvcGh5c2ljYWxfcGxhbi93aW5kb3dfZnVuY3Rpb25zLnJz) | `86.42% <ø> (+0.71%)` | :arrow_up: |
   | [datafusion/src/sql/planner.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvc3FsL3BsYW5uZXIucnM=) | `84.75% <ø> (ø)` | |
   | [datafusion/src/physical\_plan/planner.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvcGh5c2ljYWxfcGxhbi9wbGFubmVyLnJz) | `79.84% <33.33%> (+2.30%)` | :arrow_up: |
   | [datafusion/src/physical\_plan/mod.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvcGh5c2ljYWxfcGxhbi9tb2QucnM=) | `80.00% <70.96%> (+0.90%)` | :arrow_up: |
   | [datafusion/src/physical\_plan/windows.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvcGh5c2ljYWxfcGxhbi93aW5kb3dzLnJz) | `82.59% <75.00%> (-3.88%)` | :arrow_down: |
   | [...afusion/src/physical\_plan/expressions/nth\_value.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvcGh5c2ljYWxfcGxhbi9leHByZXNzaW9ucy9udGhfdmFsdWUucnM=) | `79.41% <75.67%> (-11.07%)` | :arrow_down: |
   | [datafusion/src/execution/context.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvZXhlY3V0aW9uL2NvbnRleHQucnM=) | `92.13% <100.00%> (+0.13%)` | :arrow_up: |
   | [...fusion/src/physical\_plan/expressions/row\_number.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvcGh5c2ljYWxfcGxhbi9leHByZXNzaW9ucy9yb3dfbnVtYmVyLnJz) | `94.28% <100.00%> (+13.03%)` | :arrow_up: |
   | [datafusion/src/physical\_plan/hash\_aggregate.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvcGh5c2ljYWxfcGxhbi9oYXNoX2FnZ3JlZ2F0ZS5ycw==) | `86.54% <100.00%> (ø)` | |
   | [datafusion/src/scalar.rs](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation#diff-ZGF0YWZ1c2lvbi9zcmMvc2NhbGFyLnJz) | `56.19% <100.00%> (ø)` | |
   | ... and [8 more](https://codecov.io/gh/apache/arrow-datafusion/pull/564/diff?src=pr&el=tree-more&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation) | |
   
   ------
   
   [Continue to review full report at Codecov](https://codecov.io/gh/apache/arrow-datafusion/pull/564?src=pr&el=continue&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation).
   > **Legend** - [Click here to learn more](https://docs.codecov.io/docs/codecov-delta?utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation)
   > `Δ = absolute <relative> (impact)`, `ø = not affected`, `? = missing data`
   > Powered by [Codecov](https://codecov.io/gh/apache/arrow-datafusion/pull/564?src=pr&el=footer&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation). Last update [e3e7e29...573d7b5](https://codecov.io/gh/apache/arrow-datafusion/pull/564?src=pr&el=lastupdated&utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation). Read the [comment docs](https://docs.codecov.io/docs/pull-request-comments?utm_medium=referral&utm_source=github&utm_content=comment&utm_campaign=pr+comments&utm_term=The+Apache+Software+Foundation).
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org



[GitHub] [arrow-datafusion] Jimexist commented on pull request #564: Add benchmarks to window function queries

Posted by GitBox <gi...@apache.org>.
Jimexist commented on pull request #564:
URL: https://github.com/apache/arrow-datafusion/pull/564#issuecomment-865005156






-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

For queries about this service, please contact Infrastructure at:
users@infra.apache.org