You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by GitBox <gi...@apache.org> on 2021/09/24 13:11:03 UTC

[GitHub] [incubator-doris] xy720 edited a comment on issue #6462: [Bug] Tuple is null predicate may cause be cores

xy720 edited a comment on issue #6462:
URL: https://github.com/apache/incubator-doris/issues/6462#issuecomment-926612106


   This problem is caused by SortNode incorrectly collecting TupleIsNull predicate of child nodes.
   
   explain sql:
   
   ```
   +---------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------+
   | Explain String                                                                                                                                                                                                                                                                                    |
   +---------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------+
   | PLAN FRAGMENT 0                                                                                                                                                                                                                                                                                   |
   |  OUTPUT EXPRS:<slot 57> <slot 37> `medal_data`.`event_day` | <slot 58> <slot 38> `medal_data`.`medal_level` | <slot 59> <slot 39> multi_distinct_count(DISTINCT `medal_data`.`uid`) | <slot 60> <slot 40> multi_distinct_count(DISTINCT if(`row_rank` = 1, `medal_data`.`uid`, NULL)) | <slot 56> |
   |   PARTITION: HASH_PARTITIONED: <slot 38> `medal_data`.`medal_level`                                                                                                                                                                                                                               |
   |                                                                                                                                                                                                                                                                                                   |
   |   RESULT SINK                                                                                                                                                                                                                                                                                     |
   |                                                                                                                                                                                                                                                                                                   |
   |   15:ANALYTIC                                                                                                                                                                                                                                                                                     |
   |   |  functions: [, sum(<slot 59> <slot 39> multi_distinct_count(DISTINCT `medal_data`.`uid`)), ]                                                                                                                                                                                                  |
   |   |  partition by: <slot 38> `medal_data`.`medal_level`                                                                                                                                                                                                                                           |
   |   |  order by: <slot 57> <slot 37> `medal_data`.`event_day` ASC NULLS FIRST                                                                                                                                                                                                                       |
   |   |  window: RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW                                                                                                                                                                                                                                    |
   |   |                                                                                                                                                                                                                                                                                               |
   |   14:SORT                                                                                                                                                                                                                                                                                         |
   |   |  order by: <slot 58> <slot 38> `medal_data`.`medal_level` ASC, <slot 57> <slot 37> `medal_data`.`event_day` ASC                                                                                                                                                                               |
   |   |  offset: 0                                                                                                                                                                                                                                                                                    |
   |   |                                                                                                                                                                                                                                                                                               |
   |   20:EXCHANGE                                                                                                                                                                                                                                                                                     |
   |                                                                                                                                                                                                                                                                                                   |
   | PLAN FRAGMENT 1                                                                                                                                                                                                                                                                                   |
   |  OUTPUT EXPRS:                                                                                                                                                                                                                                                                                    |
   |   PARTITION: RANDOM                                                                                                                                                                                                                                                                               |
   |                                                                                                                                                                                                                                                                                                   |
   |   STREAM DATA SINK                                                                                                                                                                                                                                                                                |
   |     EXCHANGE ID: 20                                                                                                                                                                                                                                                                               |
   |     HASH_PARTITIONED: <slot 38> `medal_data`.`medal_level`                                                                                                                                                                                                                                        |
   |                                                                                                                                                                                                                                                                                                   |
   |   13:AGGREGATE (update finalize)                                                                                                                                                                                                                                                                  |
   |   |  output: multi_distinct_count(DISTINCT if(TupleIsNull(21,20), NULL, <slot 45> <slot 14> `uid`)), multi_distinct_count(DISTINCT if(if(TupleIsNull(21,20), NULL, <slot 43>) = 1, if(TupleIsNull(21,20), NULL, <slot 45> <slot 14> `uid`), NULL))                                                |
   |   |  group by:  `event_day`, <slot 5> `medal_level`                                                                                                                                                                                                                                               |
   |   |  cardinality=-1                                                                                                                                                                                                                                                                               |
   |   |                                                                                                                                                                                                                                                                                               |
   |   12:HASH JOIN                                                                                                                                                                                                                                                                                    |
   |   |  join op: LEFT OUTER JOIN (BROADCAST)                                                                                                                                                                                                                                                         |
   |   |  hash predicates:                                                                                                                                                                                                                                                                             |
   |   |  colocate: false, reason: Src column hash been transformed by expr                                                                                                                                                                                                                            |
   |   |  equal join conjunct: if(TupleIsNull(21,20), NULL, <slot 45> <slot 14> `uid`) = <slot 29> `uid`                                                                                                                                                                                               |
   |   |  equal join conjunct:  `event_day` = <slot 30> `event_day`                                                                                                                                                                                                                                    |
   |   |  cardinality=-1                                                                                                                                                                                                                                                                               |
   |   |                                                                                                                                                                                                                                                                                               |
   |   |----19:EXCHANGE                                                                                                                                                                                                                                                                                |
   |   |                                                                                                                                                                                                                                                                                               |
   |   9:HASH JOIN                                                                                                                                                                                                                                                                                     |
   |   |  join op: LEFT OUTER JOIN (BROADCAST)                                                                                                                                                                                                                                                         |
   |   |  hash predicates:                                                                                                                                                                                                                                                                             |
   |   |  colocate: false, reason: The src data has been redistributed                                                                                                                                                                                                                                 |
   |   |  equal join conjunct:  `event_day` = <slot 44> <slot 13> `event_day`                                                                                                                                                                                                                          |
   |   |  equal join conjunct: <slot 5> `medal_level` = <slot 46> <slot 15> `medal_level`                                                                                                                                                                                                              |
   |   |  cardinality=-1                                                                                                                                                                                                                                                                               |
   |   |                                                                                                                                                                                                                                                                                               |
   |   |----18:EXCHANGE                                                                                                                                                                                                                                                                                |
   |   |                                                                                                                                                                                                                                                                                               |
   |   4:HASH JOIN                                                                                                                                                                                                                                                                                     |
   |   |  join op: INNER JOIN (BROADCAST)                                                                                                                                                                                                                                                              |
   |   |  hash predicates:                                                                                                                                                                                                                                                                             |
   |   |  colocate: false, reason: Src column hash been transformed by expr                                                                                                                                                                                                                            |
   |   |  equal join conjunct: 1 = 1                                                                                                                                                                                                                                                                   |
   |   |  cardinality=-1                                                                                                                                                                                                                                                                               |
   |   |                                                                                                                                                                                                                                                                                               |
   |   |----16:EXCHANGE                                                                                                                                                                                                                                                                                |
   |   |                                                                                                                                                                                                                                                                                               |
   |   1:AGGREGATE (update finalize)                                                                                                                                                                                                                                                                   |
   |   |  group by: `event_day`                                                                                                                                                                                                                                                                        |
   |   |  cardinality=-1                                                                                                                                                                                                                                                                               |
   |   |                                                                                                                                                                                                                                                                                               |
   |   0:OlapScanNode                                                                                                                                                                                                                                                                                  |
   |      TABLE: dwd_youjia_medal_new_incr_day                                                                                                                                                                                                                                                         |
   |      PREAGGREGATION: ON                                                                                                                                                                                                                                                                           |
   |      partitions=0/4                                                                                                                                                                                                                                                                               |
   |      rollup: null                                                                                                                                                                                                                                                                                 |
   |      tabletRatio=0/0                                                                                                                                                                                                                                                                              |
   |      tabletList=                                                                                                                                                                                                                                                                                  |
   |      cardinality=0                                                                                                                                                                                                                                                                                |
   |      avgRowSize=16.0                                                                                                                                                                                                                                                                              |
   |      numNodes=1                                                                                                                                                                                                                                                                                   |
   |                                                                                                                                                                                                                                                                                                   |
   | PLAN FRAGMENT 2                                                                                                                                                                                                                                                                                   |
   |  OUTPUT EXPRS:                                                                                                                                                                                                                                                                                    |
   |   PARTITION: RANDOM                                                                                                                                                                                                                                                                               |
   |                                                                                                                                                                                                                                                                                                   |
   |   STREAM DATA SINK                                                                                                                                                                                                                                                                                |
   |     EXCHANGE ID: 19                                                                                                                                                                                                                                                                               |
   |     UNPARTITIONED                                                                                                                                                                                                                                                                                 |
   |                                                                                                                                                                                                                                                                                                   |
   |   11:AGGREGATE (update finalize)                                                                                                                                                                                                                                                                  |
   |   |  group by: `uid`, `event_day`                                                                                                                                                                                                                                                                 |
   |   |  cardinality=-1                                                                                                                                                                                                                                                                               |
   |   |                                                                                                                                                                                                                                                                                               |
   |   10:OlapScanNode                                                                                                                                                                                                                                                                                 |
   |      TABLE: autocar_youjia_app_ubc_log_new                                                                                                                                                                                                                                                        |
   |      PREAGGREGATION: OFF. Reason: conjunct on `uid` which is StorageEngine value column                                                                                                                                                                                                           |
   |      partitions=0/2                                                                                                                                                                                                                                                                               |
   |      rollup: null                                                                                                                                                                                                                                                                                 |
   |      tabletRatio=0/0                                                                                                                                                                                                                                                                              |
   |      tabletList=                                                                                                                                                                                                                                                                                  |
   |      cardinality=0                                                                                                                                                                                                                                                                                |
   |      avgRowSize=32.0                                                                                                                                                                                                                                                                              |
   |      numNodes=1                                                                                                                                                                                                                                                                                   |
   |                                                                                                                                                                                                                                                                                                   |
   | PLAN FRAGMENT 3                                                                                                                                                                                                                                                                                   |
   |  OUTPUT EXPRS:                                                                                                                                                                                                                                                                                    |
   |   PARTITION: HASH_PARTITIONED: <slot 14> `uid`, <slot 15> `medal_level`                                                                                                                                                                                                                           |
   |                                                                                                                                                                                                                                                                                                   |
   |   STREAM DATA SINK                                                                                                                                                                                                                                                                                |
   |     EXCHANGE ID: 18                                                                                                                                                                                                                                                                               |
   |     UNPARTITIONED                                                                                                                                                                                                                                                                                 |
   |                                                                                                                                                                                                                                                                                                   |
   |   8:ANALYTIC                                                                                                                                                                                                                                                                                      |
   |   |  functions: [, row_number(), ]                                                                                                                                                                                                                                                                |
   |   |  partition by: <slot 14> `uid`, <slot 15> `medal_level`                                                                                                                                                                                                                                       |
   |   |  order by: <slot 44> <slot 13> `event_day` ASC NULLS FIRST                                                                                                                                                                                                                                    |
   |   |  window: ROWS BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW                                                                                                                                                                                                                                     |
   |   |                                                                                                                                                                                                                                                                                               |
   |   7:SORT                                                                                                                                                                                                                                                                                          |
   |   |  order by: <slot 45> <slot 14> `uid` ASC, <slot 46> <slot 15> `medal_level` ASC, <slot 44> <slot 13> `event_day` ASC                                                                                                                                                                          |
   |   |  offset: 0                                                                                                                                                                                                                                                                                    |
   |   |                                                                                                                                                                                                                                                                                               |
   |   17:EXCHANGE                                                                                                                                                                                                                                                                                     |
   |                                                                                                                                                                                                                                                                                                   |
   | PLAN FRAGMENT 4                                                                                                                                                                                                                                                                                   |
   |  OUTPUT EXPRS:                                                                                                                                                                                                                                                                                    |
   |   PARTITION: RANDOM                                                                                                                                                                                                                                                                               |
   |                                                                                                                                                                                                                                                                                                   |
   |   STREAM DATA SINK                                                                                                                                                                                                                                                                                |
   |     EXCHANGE ID: 17                                                                                                                                                                                                                                                                               |
   |     HASH_PARTITIONED: <slot 14> `uid`, <slot 15> `medal_level`                                                                                                                                                                                                                                    |
   |                                                                                                                                                                                                                                                                                                   |
   |   6:AGGREGATE (update finalize)                                                                                                                                                                                                                                                                   |
   |   |  group by: `event_day`, `uid`, `medal_level`                                                                                                                                                                                                                                                  |
   |   |  cardinality=-1                                                                                                                                                                                                                                                                               |
   |   |                                                                                                                                                                                                                                                                                               |
   |   5:OlapScanNode                                                                                                                                                                                                                                                                                  |
   |      TABLE: dwd_youjia_medal_new_incr_day                                                                                                                                                                                                                                                         |
   |      PREAGGREGATION: ON                                                                                                                                                                                                                                                                           |
   |      partitions=0/4                                                                                                                                                                                                                                                                               |
   |      rollup: null                                                                                                                                                                                                                                                                                 |
   |      tabletRatio=0/0                                                                                                                                                                                                                                                                              |
   |      tabletList=                                                                                                                                                                                                                                                                                  |
   |      cardinality=0                                                                                                                                                                                                                                                                                |
   |      avgRowSize=48.0                                                                                                                                                                                                                                                                              |
   |      numNodes=1                                                                                                                                                                                                                                                                                   |
   |                                                                                                                                                                                                                                                                                                   |
   | PLAN FRAGMENT 5                                                                                                                                                                                                                                                                                   |
   |  OUTPUT EXPRS:                                                                                                                                                                                                                                                                                    |
   |   PARTITION: RANDOM                                                                                                                                                                                                                                                                               |
   |                                                                                                                                                                                                                                                                                                   |
   |   STREAM DATA SINK                                                                                                                                                                                                                                                                                |
   |     EXCHANGE ID: 16                                                                                                                                                                                                                                                                               |
   |     UNPARTITIONED                                                                                                                                                                                                                                                                                 |
   |                                                                                                                                                                                                                                                                                                   |
   |   3:AGGREGATE (update finalize)                                                                                                                                                                                                                                                                   |
   |   |  group by: `medal_level`                                                                                                                                                                                                                                                                      |
   |   |  cardinality=-1                                                                                                                                                                                                                                                                               |
   |   |                                                                                                                                                                                                                                                                                               |
   |   2:OlapScanNode                                                                                                                                                                                                                                                                                  |
   |      TABLE: dwd_youjia_medal_new_incr_day                                                                                                                                                                                                                                                         |
   |      PREAGGREGATION: ON                                                                                                                                                                                                                                                                           |
   |      partitions=0/4                                                                                                                                                                                                                                                                               |
   |      rollup: null                                                                                                                                                                                                                                                                                 |
   |      tabletRatio=0/0                                                                                                                                                                                                                                                                              |
   |      tabletList=                                                                                                                                                                                                                                                                                  |
   |      cardinality=0                                                                                                                                                                                                                                                                                |
   |      avgRowSize=16.0                                                                                                                                                                                                                                                                              |
   |      numNodes=1                                                                                                                                                                                                                                                                                   |
   +---------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------------+
   ```
   
   The SortNode(14) has member variable `resolvedTupleExprs`, which collects output exprs from its child node, and will be sent to be.
   
   When collecting, it filters exprs not bound to tupleId of its child node through Expr.IsBoundbyTupleIds() method. (See details in AnalyticPlanner.java:createSortInfo()).
   
   Sometimes, when there is left join in the query, the child nodes in the tree will produce many TupleIsNull predicates, but we did not implement IsBoundbyTupleIds() method for TupleIsNull predicate. So SortNode will incorrectly collect wrong TupleIsNull, which may cause be core.
   


-- 
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: commits-unsubscribe@doris.apache.org

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



---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org
For additional commands, e-mail: commits-help@doris.apache.org