Uploaded image for project: 'IMPALA'
  1. IMPALA
  2. IMPALA-8790

IllegalStateException: Illegal reference to non-materialized slot

    XMLWordPrintableJSON

Details

    • Bug
    • Status: Resolved
    • Major
    • Resolution: Fixed
    • Impala 3.1.0, Impala 3.2.0
    • Impala 3.3.0
    • Frontend
    • None

    Description

      Reproduce:

      $ hdfs dfs -put foo.parq hdfs:///tmp
      impala> create table foo (uid string, cid string) stored as parquet;
      impala> load data inpath 'hdfs:///tmp/foo.parq' into table foo;
      

      With the stats, the following query hits an IllegalStateException:

      impala> compute stats foo;
      impala> explain select uid, cid,
         rank() over (partition by uid order by count(*) desc)
      from (select uid, cid from foo) w
      group by uid, cid;
      
      ERROR: IllegalStateException: Illegal reference to non-materialized slot: tid=1 sid=2

      Get the stacktrace in impalad.INFO:

      I1015 13:44:25.276245 67623 jni-util.cc:256] 474111eba829d732:233f6caa00000000] java.lang.IllegalStateException: Illegal reference to non-materialized slot: tid=1 sid=2
              at com.google.common.base.Preconditions.checkState(Preconditions.java:149)
              at org.apache.impala.analysis.SlotRef.toThrift(SlotRef.java:161)
              at org.apache.impala.analysis.Expr.treeToThriftHelper(Expr.java:752)
              at org.apache.impala.analysis.Expr.treeToThrift(Expr.java:731)
              at org.apache.impala.analysis.Expr.treesToThrift(Expr.java:783)
              at org.apache.impala.planner.DataPartition.toThrift(DataPartition.java:89)
              at org.apache.impala.planner.PlanFragment.toThrift(PlanFragment.java:338)
              at org.apache.impala.service.Frontend.createPlanExecInfo(Frontend.java:1146)
              at org.apache.impala.service.Frontend.createExecRequest(Frontend.java:1185)
              at org.apache.impala.service.Frontend.getPlannedExecRequest(Frontend.java:1466)
              at org.apache.impala.service.Frontend.doCreateExecRequest(Frontend.java:1345)
              at org.apache.impala.service.Frontend.getTExecRequest(Frontend.java:1252)
              at org.apache.impala.service.Frontend.createExecRequest(Frontend.java:1222)
              at org.apache.impala.service.JniFrontend.createExecRequest(JniFrontend.java:167)
      I1015 13:44:25.277076 67623 status.cc:124] 474111eba829d732:233f6caa00000000] IllegalStateException: Illegal reference to non-materialized slot: tid=1 sid=2 

      Without the stats, it runs successfully:

      impala> drop stats foo;
      impala> explain select uid, cid,
         rank() over (partition by uid order by count(*) desc)
      from (select uid, cid from foo) w
      group by uid, cid;
      +------------------------------------------------------------------------------------+
      | Explain String                                                                     |
      +------------------------------------------------------------------------------------+
      | Max Per-Host Resource Reservation: Memory=84.02MB Threads=5                        |
      | Per-Host Resource Estimates: Memory=304MB                                          |
      | WARNING: The following tables are missing relevant table and/or column statistics. |
      | common_action.foo                                                                  |
      |                                                                                    |
      | PLAN-ROOT SINK                                                                     |
      | |                                                                                  |
      | 07:EXCHANGE [UNPARTITIONED]                                                        |
      | |                                                                                  |
      | 03:ANALYTIC                                                                        |
      | |  functions: rank()                                                               |
      | |  partition by: uid                                                               |
      | |  order by: count(*) DESC                                                         |
      | |  window: RANGE BETWEEN UNBOUNDED PRECEDING AND CURRENT ROW                       |
      | |  row-size=40B cardinality=1.10K                                                  |
      | |                                                                                  |
      | 02:SORT                                                                            |
      | |  order by: uid ASC NULLS FIRST, count(*) DESC                                    |
      | |  row-size=32B cardinality=1.10K                                                  |
      | |                                                                                  |
      | 06:EXCHANGE [HASH(uid)]                                                            |
      | |                                                                                  |
      | 05:AGGREGATE [FINALIZE]                                                            |
      | |  output: count:merge(*)                                                          |
      | |  group by: uid, cid                                                              |
      | |  row-size=32B cardinality=1.10K                                                  |
      | |                                                                                  |
      | 04:EXCHANGE [HASH(uid,cid)]                                                        |
      | |                                                                                  |
      | 01:AGGREGATE [STREAMING]                                                           |
      | |  output: count(*)                                                                |
      | |  group by: uid, cid                                                              |
      | |  row-size=32B cardinality=1.10K                                                  |
      | |                                                                                  |
      | 00:SCAN HDFS [common_action.foo]                                                   |
      |    HDFS partitions=1/1 files=1 size=5.19KB                                         |
      |    row-size=24B cardinality=1.10K                                                  |
      +------------------------------------------------------------------------------------+
      Fetched 37 row(s) in 0.03s
      

       

      Attachments

        1. foo.parq
          5 kB
          Quanlong Huang

        Issue Links

          Activity

            People

              stigahuang Quanlong Huang
              stigahuang Quanlong Huang
              Votes:
              0 Vote for this issue
              Watchers:
              3 Start watching this issue

              Dates

                Created:
                Updated:
                Resolved: