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

to_date(to_timestamp) with group-by in hbase/maprdb table fails with "java.lang.UnsupportedOperationException"

VotersWatch issueWatchersLinkCloneUpdate Comment AuthorReplace String in CommentUpdate Comment VisibilityDelete Comments
    XMLWordPrintableJSON

Details

    • Bug
    • Status: Closed
    • Critical
    • Resolution: Fixed
    • 1.0.0
    • 1.3.0
    • Functions - Drill
    • None
    • 1.0 GA version

    Description

      1. Create a hbase/maprdb table in hbase shell:

      create '/tables/esr52','cf'
      put '/tables/esr52','1434998909','cf:c','abc'
      > scan '/tables/esr52'
      ROW                                                                  COLUMN+CELL
       1434998909                                                          column=cf:c, timestamp=1434998994785, value=abc
      

      2. Below SQLs work fine in Drill:

      >  select * from maprdb.esr52;
      +--------------+---------------+
      |   row_key    |      cf       |
      +--------------+---------------+
      | [B@5bafd971  | {"c":"YWJj"}  |
      +--------------+---------------+
      1 row selected (0.095 seconds)
      
      > select to_date(to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int))) from maprdb.esr52 esrtable;
      +-------------+
      |   EXPR$0    |
      +-------------+
      | 2015-06-22  |
      +-------------+
      1 row selected (0.127 seconds)
      

      3. However below SQL with group-by fails:

      select to_date(to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int))),count(*) from maprdb.esr52 esrtable 
      group by to_date(to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int)));
      
      Error: SYSTEM ERROR: java.lang.UnsupportedOperationException: Failure finding function that runtime code generation expected.  Signature: compare_to_nulls_high( VAR16CHAR:OPTIONAL, VAR16CHAR:OPTIONAL ) returns INT:REQUIRED
      
      Fragment 3:0
      
      [Error Id: 26003311-d40e-4a95-9d3c-68793459ad6d on h1.poc.com:31010]
      
        (java.lang.UnsupportedOperationException) Failure finding function that runtime code generation expected.  Signature: compare_to_nulls_high( VAR16CHAR:OPTIONAL, VAR16CHAR:OPTIONAL ) returns INT:REQUIRED
          org.apache.drill.exec.expr.fn.FunctionGenerationHelper.getFunctionExpression():109
          org.apache.drill.exec.expr.fn.FunctionGenerationHelper.getOrderingComparator():62
          org.apache.drill.exec.expr.fn.FunctionGenerationHelper.getOrderingComparatorNullsHigh():79
          org.apache.drill.exec.physical.impl.common.ChainedHashTable.setupIsKeyMatchInternal():257
          org.apache.drill.exec.physical.impl.common.ChainedHashTable.createAndSetupHashTable():206
          org.apache.drill.exec.test.generated.HashAggregatorGen1.setup():273
          org.apache.drill.exec.physical.impl.aggregate.HashAggBatch.createAggregatorInternal():240
          org.apache.drill.exec.physical.impl.aggregate.HashAggBatch.createAggregator():163
          org.apache.drill.exec.physical.impl.aggregate.HashAggBatch.buildSchema():110
          org.apache.drill.exec.record.AbstractRecordBatch.next():127
          org.apache.drill.exec.record.AbstractRecordBatch.next():105
          org.apache.drill.exec.record.AbstractRecordBatch.next():95
          org.apache.drill.exec.record.AbstractSingleRecordBatch.innerNext():51
          org.apache.drill.exec.physical.impl.project.ProjectRecordBatch.innerNext():129
          org.apache.drill.exec.record.AbstractRecordBatch.next():146
          org.apache.drill.exec.physical.impl.BaseRootExec.next():83
          org.apache.drill.exec.physical.impl.SingleSenderCreator$SingleSenderRootExec.innerNext():95
          org.apache.drill.exec.physical.impl.BaseRootExec.next():73
          org.apache.drill.exec.work.fragment.FragmentExecutor$1.run():259
          org.apache.drill.exec.work.fragment.FragmentExecutor$1.run():253
          java.security.AccessController.doPrivileged():-2
          javax.security.auth.Subject.doAs():422
          org.apache.hadoop.security.UserGroupInformation.doAs():1566
          org.apache.drill.exec.work.fragment.FragmentExecutor.run():253
          org.apache.drill.common.SelfCleaningRunnable.run():38
          java.util.concurrent.ThreadPoolExecutor.runWorker():1142
          java.util.concurrent.ThreadPoolExecutor$Worker.run():617
          java.lang.Thread.run():745 (state=,code=0)
      

      4. If we remove to_date, and only group-by to_timestamp, it works fine:

      select to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int)) from maprdb.esr52 esrtable;
      +------------------------+
      |         EXPR$0         |
      +------------------------+
      | 2015-06-22 18:48:29.0  |
      +------------------------+
      1 row selected (0.084 seconds)
      
      select to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int)),count(*) from maprdb.esr52 esrtable
      group by to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int));
      +------------------------+---------+
      |         EXPR$0         | EXPR$1  |
      +------------------------+---------+
      | 2015-06-22 18:48:29.0  | 1       |
      +------------------------+---------+
      1 row selected (0.641 seconds)
      

      As a workaround, we can use substr instead of to_date.

      select substr(to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int)),1,10) from maprdb.esr52 esrtable;
      +-------------+
      |   EXPR$0    |
      +-------------+
      | 2015-06-22  |
      +-------------+
      1 row selected (0.132 seconds)
      
      select substr(to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int)),1,10),count(*) from maprdb.esr52 esrtable
      group by substr(to_timestamp(cast(convert_from(esrtable.row_key,'UTF8') as int)),1,10);
      +-------------+---------+
      |   EXPR$0    | EXPR$1  |
      +-------------+---------+
      | 2015-06-22  | 1       |
      +-------------+---------+
      1 row selected (0.678 seconds)
      

      Need fix.

      Attachments

        Issue Links

        Activity

          This comment will be Viewable by All Users Viewable by All Users
          Cancel

          People

            mehant Mehant Baid
            haozhu Hao Zhu
            Abhishek Girish Abhishek Girish
            Votes:
            0 Vote for this issue
            Watchers:
            3 Start watching this issue

            Dates

              Created:
              Updated:
              Resolved:

              Slack

                Issue deployment