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 2022/11/03 11:49:03 UTC

[GitHub] [arrow-datafusion] yukkit opened a new issue, #4096: Panic when the number of partitions of the pipeline that throws the exception is inconsistent with the number of partitions output by the query

yukkit opened a new issue, #4096:
URL: https://github.com/apache/arrow-datafusion/issues/4096

   **Describe the bug**
   
   I am the developer of CnosDB, when we are doing integration testing, some SQL causes panic.
   
   such as:
   ```sql
   WITH gp AS (SELECT id FROM table1 GROUP BY id) 
   SELECT
       COUNT(CAST(CAST(gp.id || '' AS TIMESTAMP) AS BIGINT)) 
   FROM 
       gp;
   ```
   
   **To Reproduce**
   
   1. physical plan
   
   ```
   Plan: ProjectionExec: expr=[COUNT(gp.id || Utf8(""))@0 as COUNT(gp.id || Utf8(""))]
     AggregateExec: mode=Final, gby=[], aggr=[COUNT(gp.id || Utf8(""))]
       CoalescePartitionsExec
         AggregateExec: mode=Partial, gby=[], aggr=[COUNT(gp.id || Utf8(""))]
           ProjectionExec: expr=[id@0 as id]
             AggregateExec: mode=FinalPartitioned, gby=[id@0 as id], aggr=[]
               CoalesceBatchesExec: target_batch_size=4096
                 RepartitionExec: partitioning=Hash([Column { name: "id", index: 0 }], 4)
                   AggregateExec: mode=Partial, gby=[id@0 as id], aggr=[]
                     RepartitionExec: partitioning=RoundRobinBatch(4)
                       MemoryExec: partitions=2, partition_sizes=[20, 20]
   ```
   
   2. result
   
   ```
   thread 'df-worker-1' panicked at 'index out of bounds: the len is 1 but the index is 2', datafusion/core/src/scheduler/task.rs:306:17
   note: run with `RUST_BACKTRACE=1` environment variable to display a backtrace
   thread 'df-worker-2' panicked at 'index out of bounds: the len is 1 but the index is 1', datafusion/core/src/scheduler/task.rs:306:17
   thread 'df-worker-3' panicked at 'index out of bounds: the len is 1 but the index is 3', datafusion/core/src/scheduler/task.rs:306:17
   thread 'scheduler::tests::test_runtime_err' panicked at 'called `Result::unwrap()` on an `Err` value: ExternalError(ArrowError(ExternalError(ArrowError(CastError("Error parsing '2' as timestamp")))))', datafusion/core/src/scheduler/mod.rs:480:64
   [2022-11-03T08:53:53Z ERROR datafusion::scheduler] worker 3 panicked with: index out of bounds: the len is 1 but the index is 3
   [2022-11-03T08:53:53Z ERROR datafusion::scheduler] worker 2 panicked with: index out of bounds: the len is 1 but the index is 1
   ```
   
   **Expected behavior**
   no panic
   
   **Additional context**
   
   I am trying to fix this


-- 
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.

To unsubscribe, e-mail: github-unsubscribe@arrow.apache.org.apache.org

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


[GitHub] [arrow-datafusion] yukkit commented on issue #4096: Panic when the number of partitions of the pipeline that throws the exception is inconsistent with the number of partitions output by the query

Posted by GitBox <gi...@apache.org>.
yukkit commented on issue #4096:
URL: https://github.com/apache/arrow-datafusion/issues/4096#issuecomment-1301987391

   I use the **Scheduler** to execute execution plan.


-- 
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.

To unsubscribe, e-mail: github-unsubscribe@arrow.apache.org

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


[GitHub] [arrow-datafusion] tustvold closed issue #4096: Panic when the number of partitions of the pipeline that throws the exception is inconsistent with the number of partitions output by the query

Posted by GitBox <gi...@apache.org>.
tustvold closed issue #4096: Panic when the number of partitions of the pipeline that throws the exception is inconsistent with the number of partitions output by the query
URL: https://github.com/apache/arrow-datafusion/issues/4096


-- 
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.

To unsubscribe, e-mail: github-unsubscribe@arrow.apache.org

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