Uploaded image for project: 'Apache Drill'
  1. Apache Drill
  2. DRILL-1888

Memory leak for aggregation with HAVING clause

    XMLWordPrintableJSON

Details

    Description

      Use TPC-DS scema and run query below:

      SELECT
              wr_returning_customer_sk,
              wr_returned_date_sk,
              sum(wr_return_amt)/sum(wr_return_quantity) as avg_return_amt
      FROM web_returns
      WHERE (
              wr_item_sk in   (
                              SELECT
                                      ws_item_sk
                              FROM    web_sales
                              WHERE ws_sales_price < 50 or ws_ext_sales_price < 1000)
              or wr_item_sk in (1, 2, 3, 4, 5, 6, 7, 8, 9, 10) )
              and wr_reason_sk in (10)
      GROUP BY
              wr_returning_customer_sk,
              wr_returned_date_sk
      HAVING sum(wr_return_amt)/sum(wr_return_quantity) >= 50.0;
      

      From drillbit.log

      Total 1 allocation(s) of byte size(s): 4096, at stack location:
      org.apache.drill.exec.memory.TopLevelAllocator$ChildAllocator.buffer(TopLevelAllocator.java:212)
                      org.apache.drill.exec.vector.UInt1Vector.allocateNewSafe(UInt1Vector.java:137)
                      org.apache.drill.exec.vector.UInt1Vector.allocateNew(UInt1Vector.java:123)
                      org.apache.drill.exec.vector.NullableIntVector.allocateNew(NullableIntVector.java:163)
                      org.apache.drill.exec.physical.impl.aggregate.HashAggBatch.buildSchema(HashAggBatch.java:101)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:130)
                      org.apache.drill.exec.physical.impl.validate.IteratorValidatorBatchIterator.next(IteratorValidatorBatchIterator.java:118)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:99)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:89)
                      org.apache.drill.exec.record.AbstractSingleRecordBatch.innerNext(AbstractSingleRecordBatch.java:51)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:142)
                      org.apache.drill.exec.physical.impl.validate.IteratorValidatorBatchIterator.next(IteratorValidatorBatchIterator.java:118)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:99)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:89)
                      org.apache.drill.exec.record.AbstractSingleRecordBatch.innerNext(AbstractSingleRecordBatch.java:51)
                      org.apache.drill.exec.physical.impl.svremover.RemovingRecordBatch.innerNext(RemovingRecordBatch.java:96)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:142)
                      org.apache.drill.exec.physical.impl.validate.IteratorValidatorBatchIterator.next(IteratorValidatorBatchIterator.java:118)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:99)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:89)
                      org.apache.drill.exec.record.AbstractSingleRecordBatch.innerNext(AbstractSingleRecordBatch.java:51)
                      org.apache.drill.exec.physical.impl.project.ProjectRecordBatch.innerNext(ProjectRecordBatch.java:132)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:142)
                      org.apache.drill.exec.physical.impl.validate.IteratorValidatorBatchIterator.next(IteratorValidatorBatchIterator.java:118)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:99)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:89)
                      org.apache.drill.exec.record.AbstractSingleRecordBatch.innerNext(AbstractSingleRecordBatch.java:51)
                      org.apache.drill.exec.physical.impl.project.ProjectRecordBatch.innerNext(ProjectRecordBatch.java:132)
                      org.apache.drill.exec.record.AbstractRecordBatch.next(AbstractRecordBatch.java:142)
                      org.apache.drill.exec.physical.impl.validate.IteratorValidatorBatchIterator.next(IteratorValidatorBatchIterator.java:118)
                      org.apache.drill.exec.physical.impl.BaseRootExec.next(BaseRootExec.java:67)
                      org.apache.drill.exec.physical.impl.ScreenCreator$ScreenRoot.innerNext(ScreenCreator.java:97)
                      org.apache.drill.exec.physical.impl.BaseRootExec.next(BaseRootExec.java:57)
                      org.apache.drill.exec.work.fragment.FragmentExecutor.run(FragmentExecutor.java:114)
                      org.apache.drill.exec.work.WorkManager$RunnableWrapper.run(WorkManager.java:254)
                      java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1145)
                      java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:615)
                      java.lang.Thread.run(Thread.java:745)
      

      Attachments

        Activity

          People

            amansinha100 Aman Sinha
            vicky Victoria Markman
            Votes:
            0 Vote for this issue
            Watchers:
            4 Start watching this issue

            Dates

              Created:
              Updated:
              Resolved: