Skip to content
New issue

Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.

By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.

Already on GitHub? Sign in to your account

frontend stack overflow when calling to_stream_prost on a complex query #15916

Closed
st1page opened this issue Mar 26, 2024 · 3 comments · Fixed by #16279
Closed

frontend stack overflow when calling to_stream_prost on a complex query #15916

st1page opened this issue Mar 26, 2024 · 3 comments · Fixed by #16279
Assignees
Milestone

Comments

@st1page
Copy link
Contributor

st1page commented Mar 26, 2024

https://buildkite.com/risingwavelabs/pull-request/builds/45532#018e79dc-9596-4f36-9aed-c57fabef65ce

It seems to be caused by

SET RW_FORCE_TWO_PHASE_AGG TO TRUE
CREATE MATERIALIZED VIEW stream_query AS SELECT tumble_10.c6 AS col_0, tumble_10.c11 AS col_1, (TIMESTAMP '2024-03-26 08:35:02') AS col_2 FROM ((SELECT t_0.c6 AS col_0, ((SMALLINT '232')) AS col_1, t_1.col_2 AS col_2, t_0.c6 AS col_3 FROM (alltypes2 AS t_0 RIGHT JOIN m6 AS t_1 ON t_0.c9 = t_1.col_0) LEFT JOIN (SELECT sq_6.col_2 AS col_0, sq_6.col_2 AS col_1, (INT '818') AS col_2 FROM (SELECT TIME '08:33:59' AS col_0, ((FLOAT '852')) AS col_1, TIMESTAMP '2024-03-26 08:34:13' AS col_2, (INT '973') AS col_3 FROM (SELECT tumble_2.state AS col_0, (coalesce(NULL, NULL, NULL, NULL, NULL, NULL, NULL, (SMALLINT '-32768'), NULL, NULL)) AS col_1, tumble_2.state AS col_2, tumble_2.state AS col_3 FROM tumble(person, person.date_time, INTERVAL '5') AS tumble_2 FULL JOIN tumble(alltypes1, alltypes1.c11, INTERVAL '17') AS tumble_3 ON tumble_2.credit_card = tumble_3.c9 WHERE tumble_3.c1 GROUP BY tumble_2.state HAVING true) AS sq_4 LEFT JOIN bid AS t_5 ON sq_4.col_0 = t_5.url) AS sq_6 FULL JOIN nation AS t_7 ON sq_6.col_3 = t_7.n_nationkey AND true WHERE false GROUP BY sq_6.col_2 HAVING false) AS sq_8 ON t_0.c11 = sq_8.col_1 AND true WHERE t_0.c1 GROUP BY t_0.c6, t_1.col_2 HAVING true) AS sq_9 FULL JOIN tumble(alltypes1, alltypes1.c11, INTERVAL '18') AS tumble_10 ON sq_9.col_2 = tumble_10.c9 AND true) LEFT JOIN tumble(bid, bid.date_time, INTERVAL '3') AS tumble_11 ON tumble_10.c4 = tumble_11.price AND tumble_10.c1 WHERE false GROUP BY tumble_10.c6, tumble_10.c11
@st1page
Copy link
Contributor Author

st1page commented Mar 26, 2024

reproduce

CREATE TABLE supplier (s_suppkey INT, s_name CHARACTER VARYING, s_address CHARACTER VARYING, s_nationkey INT, s_phone CHARACTER VARYING, s_acctbal NUMERIC, s_comment CHARACTER VARYING, PRIMARY KEY (s_suppkey));
CREATE TABLE part (p_partkey INT, p_name CHARACTER VARYING, p_mfgr CHARACTER VARYING, p_brand CHARACTER VARYING, p_type CHARACTER VARYING, p_size INT, p_container CHARACTER VARYING, p_retailprice NUMERIC, p_comment CHARACTER VARYING, PRIMARY KEY (p_partkey));
CREATE TABLE partsupp (ps_partkey INT, ps_suppkey INT, ps_availqty INT, ps_supplycost NUMERIC, ps_comment CHARACTER VARYING, PRIMARY KEY (ps_partkey, ps_suppkey));
CREATE TABLE customer (c_custkey INT, c_name CHARACTER VARYING, c_address CHARACTER VARYING, c_nationkey INT, c_phone CHARACTER VARYING, c_acctbal NUMERIC, c_mktsegment CHARACTER VARYING, c_comment CHARACTER VARYING, PRIMARY KEY (c_custkey));
CREATE TABLE orders (o_orderkey BIGINT, o_custkey INT, o_orderstatus CHARACTER VARYING, o_totalprice NUMERIC, o_orderdate DATE, o_orderpriority CHARACTER VARYING, o_clerk CHARACTER VARYING, o_shippriority INT, o_comment CHARACTER VARYING, PRIMARY KEY (o_orderkey));
CREATE TABLE lineitem (l_orderkey BIGINT, l_partkey INT, l_suppkey INT, l_linenumber INT, l_quantity NUMERIC, l_extendedprice NUMERIC, l_discount NUMERIC, l_tax NUMERIC, l_returnflag CHARACTER VARYING, l_linestatus CHARACTER VARYING, l_shipdate DATE, l_commitdate DATE, l_receiptdate DATE, l_shipinstruct CHARACTER VARYING, l_shipmode CHARACTER VARYING, l_comment CHARACTER VARYING, PRIMARY KEY (l_orderkey, l_linenumber));
CREATE TABLE nation (n_nationkey INT, n_name CHARACTER VARYING, n_regionkey INT, n_comment CHARACTER VARYING, PRIMARY KEY (n_nationkey));
CREATE TABLE region (r_regionkey INT, r_name CHARACTER VARYING, r_comment CHARACTER VARYING, PRIMARY KEY (r_regionkey));
CREATE TABLE person (id BIGINT, name CHARACTER VARYING, email_address CHARACTER VARYING, credit_card CHARACTER VARYING, city CHARACTER VARYING, state CHARACTER VARYING, date_time TIMESTAMP, extra CHARACTER VARYING, PRIMARY KEY (id));
CREATE TABLE auction (id BIGINT, item_name CHARACTER VARYING, description CHARACTER VARYING, initial_bid BIGINT, reserve BIGINT, date_time TIMESTAMP, expires TIMESTAMP, seller BIGINT, category BIGINT, extra CHARACTER VARYING, PRIMARY KEY (id));
CREATE TABLE bid (auction BIGINT, bidder BIGINT, price BIGINT, channel CHARACTER VARYING, url CHARACTER VARYING, date_time TIMESTAMP, extra CHARACTER VARYING);
CREATE TABLE alltypes1 (c1 BOOLEAN, c2 SMALLINT, c3 INT, c4 BIGINT, c5 REAL, c6 DOUBLE, c7 NUMERIC, c8 DATE, c9 CHARACTER VARYING, c10 TIME, c11 TIMESTAMP, c13 INTERVAL, c14 STRUCT<a INT>, c15 INT[], c16 CHARACTER VARYING[]);
CREATE TABLE alltypes2 (c1 BOOLEAN, c2 SMALLINT, c3 INT, c4 BIGINT, c5 REAL, c6 DOUBLE, c7 NUMERIC, c8 DATE, c9 CHARACTER VARYING, c10 TIME, c11 TIMESTAMP, c13 INTERVAL, c14 STRUCT<a INT>, c15 INT[], c16 CHARACTER VARYING[]);
CREATE MATERIALIZED VIEW m6 AS SELECT t_5.r_comment AS col_0, ((SMALLINT '1') * (((FLOAT '0') + (FLOAT '113')) * (INTERVAL '-604800'))) AS col_1, t_5.r_comment AS col_2 FROM region AS t_5 WHERE false GROUP BY t_5.r_comment;

CREATE MATERIALIZED VIEW stream_query AS
SELECT
    tumble_10.c6 AS col_0,
    tumble_10.c11 AS col_1,
    (TIMESTAMP '2024-03-26 08:35:02') AS col_2
FROM
    (
        (
            SELECT
                t_0.c6 AS col_0,
                ((SMALLINT '232')) AS col_1,
                t_1.col_2 AS col_2,
                t_0.c6 AS col_3
            FROM
                (
                    alltypes2 AS t_0
                    RIGHT JOIN m6 AS t_1 ON t_0.c9 = t_1.col_0
                )
                LEFT JOIN (
                    SELECT
                        sq_6.col_2 AS col_0,
                        sq_6.col_2 AS col_1,
                        (INT '818') AS col_2
                    FROM
                        (
                            SELECT
                                TIME '08:33:59' AS col_0,
                                ((FLOAT '852')) AS col_1,
                                TIMESTAMP '2024-03-26 08:34:13' AS col_2,
                                (INT '973') AS col_3
                            FROM
                                (
                                    SELECT
                                        tumble_2.state AS col_0,
                                        (
                                            coalesce(
                                                NULL,
                                                NULL,
                                                NULL,
                                                NULL,
                                                NULL,
                                                NULL,
                                                NULL,
                                                (SMALLINT '-32768'),
                                                NULL,
                                                NULL
                                            )
                                        ) AS col_1,
                                        tumble_2.state AS col_2,
                                        tumble_2.state AS col_3
                                    FROM
                                        tumble(person, person.date_time, INTERVAL '5') AS tumble_2 FULL
                                        JOIN tumble(alltypes1, alltypes1.c11, INTERVAL '17') AS tumble_3 ON tumble_2.credit_card = tumble_3.c9
                                    WHERE
                                        tumble_3.c1
                                    GROUP BY
                                        tumble_2.state
                                    HAVING
                                        true
                                ) AS sq_4
                                LEFT JOIN bid AS t_5 ON sq_4.col_0 = t_5.url
                        ) AS sq_6 FULL
                        JOIN nation AS t_7 ON sq_6.col_3 = t_7.n_nationkey
                        AND true
                    WHERE
                        false
                    GROUP BY
                        sq_6.col_2
                    HAVING
                        false
                ) AS sq_8 ON t_0.c11 = sq_8.col_1
                AND true
            WHERE
                t_0.c1
            GROUP BY
                t_0.c6,
                t_1.col_2
            HAVING
                true
        ) AS sq_9 FULL
        JOIN tumble(alltypes1, alltypes1.c11, INTERVAL '18') AS tumble_10 ON sq_9.col_2 = tumble_10.c9
        AND true
    )
    LEFT JOIN tumble(bid, bid.date_time, INTERVAL '3') AS tumble_11 ON tumble_10.c4 = tumble_11.price
    AND tumble_10.c1
WHERE
    false
GROUP BY
    tumble_10.c6,
    tumble_10.c11;

@st1page
Copy link
Contributor Author

st1page commented Mar 26, 2024

The explain runs well but explain (distsql) fails so the error happens in fragmentor

                                                                       QUERY PLAN
--------------------------------------------------------------------------------------------------------------------------------------------------------
 StreamMaterialize { columns: [col_0, col_1, col_2], stream_key: [col_0, col_1], pk_columns: [col_0, col_1], pk_conflict: NoCheck }
 └─StreamProject { exprs: [alltypes1.c6, alltypes1.c11, '2024-03-26 08:35:02':Timestamp] }
   └─StreamHashAgg { group_key: [alltypes1.c6, alltypes1.c11], aggs: [count] }
     └─StreamExchange { dist: HashShard(alltypes1.c6, alltypes1.c11) }
       └─StreamHashJoin { type: LeftOuter, predicate: alltypes1.c4 = bid.price AND alltypes1.c1 }
         ├─StreamExchange { dist: HashShard(alltypes1.c4) }
         │ └─StreamFilter { predicate: false:Boolean }
         │   └─StreamFilter { predicate: ((IsNotNull(alltypes2.c6) OR IsNotNull(m6.col_2)) OR IsNotNull(alltypes1._row_id)) }
         │     └─StreamHashJoin { type: FullOuter, predicate: m6.col_2 = alltypes1.c9 }
         │       ├─StreamExchange { dist: HashShard(m6.col_2) }
         │       │ └─StreamProject { exprs: [m6.col_2, alltypes2.c6] }
         │       │   └─StreamHashAgg { group_key: [alltypes2.c6, m6.col_2], aggs: [count] }
         │       │     └─StreamExchange { dist: HashShard(alltypes2.c6, m6.col_2) }
         │       │       └─StreamHashJoin { type: LeftOuter, predicate: alltypes2.c11 = '2024-03-26 08:34:13':Timestamp }
         │       │         ├─StreamExchange { dist: HashShard(alltypes2.c11) }
         │       │         │ └─StreamFilter { predicate: alltypes2.c1 }
         │       │         │   └─StreamHashJoin { type: LeftOuter, predicate: m6.col_0 = alltypes2.c9 }
         │       │         │     ├─StreamExchange { dist: HashShard(m6.col_0) }
         │       │         │     │ └─StreamTableScan { table: m6, columns: [col_0, col_2] }
         │       │         │     └─StreamExchange { dist: HashShard(alltypes2.c9) }
         │       │         │       └─StreamTableScan { table: alltypes2, columns: [c1, c6, c9, c11, _row_id] }
         │       │         └─StreamProject { exprs: ['2024-03-26 08:34:13':Timestamp] }
         │       │           └─StreamHashAgg { group_key: ['2024-03-26 08:34:13':Timestamp], aggs: [count] }
         │       │             └─StreamExchange { dist: HashShard('2024-03-26 08:34:13':Timestamp) }
         │       │               └─StreamFilter { predicate: false:Boolean }
         │       │                 └─StreamFilter { predicate: ((IsNotNull(person.state) OR IsNotNull(bid._row_id)) OR IsNotNull(nation.n_nationkey)) }
         │       │                   └─StreamHashJoin { type: FullOuter, predicate: 973:Int32 = nation.n_nationkey }
         │       │                     ├─StreamExchange { dist: HashShard(973:Int32) }
         │       │                     │ └─StreamProject { exprs: ['2024-03-26 08:34:13':Timestamp, 973:Int32, person.state, bid._row_id] }
         │       │                     │   └─StreamHashJoin { type: LeftOuter, predicate: person.state = bid.url }
         │       │                     │     ├─StreamProject { exprs: [person.state] }
         │       │                     │     │ └─StreamHashAgg { group_key: [person.state], aggs: [count] }
         │       │                     │     │   └─StreamExchange { dist: HashShard(person.state) }
         │       │                     │     │     └─StreamFilter { predicate: alltypes1.c1 }
         │       │                     │     │       └─StreamFilter { predicate: (IsNotNull(person.id) OR IsNotNull(alltypes1._row_id)) }
         │       │                     │     │         └─StreamHashJoin { type: FullOuter, predicate: person.credit_card = alltypes1.c9 }
         │       │                     │     │           ├─StreamExchange { dist: HashShard(person.credit_card) }
         │       │                     │     │           │ └─StreamTableScan { table: person, columns: [credit_card, state, id] }
         │       │                     │     │           └─StreamExchange { dist: HashShard(alltypes1.c9) }
         │       │                     │     │             └─StreamTableScan { table: alltypes1, columns: [c1, c9, _row_id] }
         │       │                     │     └─StreamExchange { dist: HashShard(bid.url) }
         │       │                     │       └─StreamTableScan { table: bid, columns: [url, _row_id] }
         │       │                     └─StreamExchange { dist: HashShard(nation.n_nationkey) }
         │       │                       └─StreamTableScan { table: nation, columns: [n_nationkey] }
         │       └─StreamExchange { dist: HashShard(alltypes1.c9) }
         │         └─StreamTableScan { table: alltypes1, columns: [c1, c4, c6, c9, c11, _row_id] }
         └─StreamExchange { dist: HashShard(bid.price) }
           └─StreamFilter { predicate: false:Boolean }
             └─StreamTableScan { table: bid, columns: [price, _row_id] }

@BugenZhao BugenZhao self-assigned this Mar 28, 2024
@BugenZhao
Copy link
Member

It's not the fault of fragmentor. The stack overflows during the recursive calls of to_stream_prost.

let input = self
.inputs()
.into_iter()
.map(|plan| plan.to_stream_prost(state))
.try_collect()?;

It is also not a bug of infinite recursion. When I increase the stack size to 1GB (with env var RUST_MIN_STACK), the materialized view is created successfully. Therefore, the reason might simply be that the plan is too complex.

Some possible improvements:

  • Limit the recursion depth and throw an error gracefully in this case.
  • Support growing the stack during runtime.
  • Rewrite the code logic in optimizer to replace recursion with iteration as much as possible.

@BugenZhao BugenZhao changed the title bug: frontend stack overflow. frontend stack overflow when calling to_stream_prost on a complex query Apr 9, 2024
Sign up for free to join this conversation on GitHub. Already have an account? Sign in to comment
Labels
None yet
Projects
None yet
Development

Successfully merging a pull request may close this issue.

2 participants