You are viewing a plain text version of this content. The canonical link for it is here.
Posted to dev@hive.apache.org by "Igor Bezuglyi (JIRA)" <ji...@apache.org> on 2019/05/07 20:17:00 UTC

[jira] [Created] (HIVE-21702) Queries with filter on a partition column fail when using standalone metastore

Igor Bezuglyi created HIVE-21702:
------------------------------------

             Summary: Queries with filter on a partition column fail when using standalone metastore
                 Key: HIVE-21702
                 URL: https://issues.apache.org/jira/browse/HIVE-21702
             Project: Hive
          Issue Type: Bug
          Components: HiveServer2, Standalone Metastore
    Affects Versions: 3.1.0
         Environment: HDP 3.1 on AWS (Hive 3.1.0)
Standalone metastore on an EC2 instance (using hive-standalone-metastore-3.1.0.3.1.0.0-78.jar)
            Reporter: Igor Bezuglyi
         Attachments: table_data_a.csv

Queries with filter on a partition column fail when using standalone metastore

Example (Hiveserver2 is pointing to a standalone metastore instance)
{{create table test_a_src (id integer, value varchar(255), run_id varchar(14)) }}{{row format delimited fields terminated by ','}}{{stored as textfile;}}

{{create table test_a (id integer, value varchar(255)) }}{{partitioned by (run_id varchar(14))}}{{row format delimited fields terminated by ','}}{{stored as textfile;}}

{{<Upload a csv file to test_a_src data directory - test file is }}{{attached>}}

{{insert into test_a select * from test_a_src;}}

{{select id from test_a where run_id = '20190502123500';}}
{{Error: Error while compiling statement: FAILED: SemanticException MetaException(message:java.lang.UnsupportedOperationException) (state=42000,code=40000)}}

The same queries work fine when using the embedded metastore. Filters on any other column work fine as well. The problem is with partition columns.

Here's an excerpt from hiveserver2.log:

{{2019-05-07T18:57:20,552 WARN [HiveServer2-Handler-Pool: Thread-474]: thrift.ThriftCLIService (:()) - Error executing statement: }}
{{org.apache.hive.service.cli.HiveSQLException: Error while compiling statement: FAILED: SemanticException MetaException(message:java.lang.UnsupportedOperationException)}}
{{ at org.apache.hive.service.cli.operation.Operation.toSQLException(Operation.java:335) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.operation.SQLOperation.prepare(SQLOperation.java:199) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.operation.SQLOperation.runInternal(SQLOperation.java:262) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.operation.Operation.run(Operation.java:247) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.session.HiveSessionImpl.executeStatementInternal(HiveSessionImpl.java:575) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.session.HiveSessionImpl.executeStatementAsync(HiveSessionImpl.java:561) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.CLIService.executeStatementAsync(CLIService.java:315) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.thrift.ThriftCLIService.ExecuteStatement(ThriftCLIService.java:566) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.rpc.thrift.TCLIService$Processor$ExecuteStatement.getResult(TCLIService.java:1557) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.rpc.thrift.TCLIService$Processor$ExecuteStatement.getResult(TCLIService.java:1542) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.thrift.ProcessFunction.process(ProcessFunction.java:39) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.thrift.TBaseProcessor.process(TBaseProcessor.java:39) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.auth.TSetIpAddressProcessor.process(TSetIpAddressProcessor.java:56) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.thrift.server.TThreadPoolServer$WorkerProcess.run(TThreadPoolServer.java:286) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1142) ~[?:1.8.0_112]}}
{{ at java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:617) ~[?:1.8.0_112]}}
{{ at java.lang.Thread.run(Thread.java:745) [?:1.8.0_112]}}
{{Caused by: org.apache.hadoop.hive.ql.parse.SemanticException: MetaException(message:java.lang.UnsupportedOperationException)}}
{{ at org.apache.hadoop.hive.ql.optimizer.pcr.PcrOpProcFactory$FilterPCR.process(PcrOpProcFactory.java:114) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultRuleDispatcher.dispatch(DefaultRuleDispatcher.java:90) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.dispatchAndReturn(DefaultGraphWalker.java:105) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.dispatch(DefaultGraphWalker.java:89) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.walk(DefaultGraphWalker.java:158) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.startWalking(DefaultGraphWalker.java:120) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.pcr.PartitionConditionRemover.transform(PartitionConditionRemover.java:86) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.Optimizer.optimize(Optimizer.java:250) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.SemanticAnalyzer.analyzeInternal(SemanticAnalyzer.java:12423) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.CalcitePlanner.analyzeInternal(CalcitePlanner.java:360) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.BaseSemanticAnalyzer.analyze(BaseSemanticAnalyzer.java:289) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compile(Driver.java:664) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileInternal(Driver.java:1869) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileAndRespond(Driver.java:1816) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileAndRespond(Driver.java:1811) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.reexec.ReExecDriver.compileAndRespond(ReExecDriver.java:126) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.operation.SQLOperation.prepare(SQLOperation.java:197) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ ... 15 more}}
{{Caused by: org.apache.hadoop.hive.ql.parse.SemanticException: MetaException(message:java.lang.UnsupportedOperationException)}}
{{ at org.apache.hadoop.hive.ql.optimizer.ppr.PartitionPruner.getPartitionsFromServer(PartitionPruner.java:474) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.ppr.PartitionPruner.prune(PartitionPruner.java:227) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.ppr.PartitionPruner.prune(PartitionPruner.java:147) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.ParseContext.getPrunedPartitions(ParseContext.java:535) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.pcr.PcrOpProcFactory$FilterPCR.process(PcrOpProcFactory.java:110) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultRuleDispatcher.dispatch(DefaultRuleDispatcher.java:90) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.dispatchAndReturn(DefaultGraphWalker.java:105) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.dispatch(DefaultGraphWalker.java:89) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.walk(DefaultGraphWalker.java:158) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.startWalking(DefaultGraphWalker.java:120) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.pcr.PartitionConditionRemover.transform(PartitionConditionRemover.java:86) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.Optimizer.optimize(Optimizer.java:250) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.SemanticAnalyzer.analyzeInternal(SemanticAnalyzer.java:12423) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.CalcitePlanner.analyzeInternal(CalcitePlanner.java:360) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.BaseSemanticAnalyzer.analyze(BaseSemanticAnalyzer.java:289) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compile(Driver.java:664) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileInternal(Driver.java:1869) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileAndRespond(Driver.java:1816) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileAndRespond(Driver.java:1811) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.reexec.ReExecDriver.compileAndRespond(ReExecDriver.java:126) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.operation.SQLOperation.prepare(SQLOperation.java:197) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ ... 15 more}}
{{Caused by: org.apache.hadoop.hive.metastore.api.MetaException: java.lang.UnsupportedOperationException}}
{{ at org.apache.hadoop.hive.metastore.api.ThriftHiveMetastore$get_partitions_by_expr_result$get_partitions_by_expr_resultStandardScheme.read(ThriftHiveMetastore.java) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.metastore.api.ThriftHiveMetastore$get_partitions_by_expr_result$get_partitions_by_expr_resultStandardScheme.read(ThriftHiveMetastore.java) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.metastore.api.ThriftHiveMetastore$get_partitions_by_expr_result.read(ThriftHiveMetastore.java) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.thrift.TServiceClient.receiveBase(TServiceClient.java:86) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.metastore.api.ThriftHiveMetastore$Client.recv_get_partitions_by_expr(ThriftHiveMetastore.java:3356) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.metastore.api.ThriftHiveMetastore$Client.get_partitions_by_expr(ThriftHiveMetastore.java:3343) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.metastore.HiveMetaStoreClient.listPartitionsByExpr(HiveMetaStoreClient.java:1584) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.metastore.HiveMetaStoreClient.listPartitionsByExpr(HiveMetaStoreClient.java:1565) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at sun.reflect.GeneratedMethodAccessor124.invoke(Unknown Source) ~[?:?]}}
{{ at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) ~[?:1.8.0_112]}}
{{ at java.lang.reflect.Method.invoke(Method.java:498) ~[?:1.8.0_112]}}
{{ at org.apache.hadoop.hive.metastore.RetryingMetaStoreClient.invoke(RetryingMetaStoreClient.java:212) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at com.sun.proxy.$Proxy37.listPartitionsByExpr(Unknown Source) ~[?:?]}}
{{ at sun.reflect.GeneratedMethodAccessor124.invoke(Unknown Source) ~[?:?]}}
{{ at sun.reflect.DelegatingMethodAccessorImpl.invoke(DelegatingMethodAccessorImpl.java:43) ~[?:1.8.0_112]}}
{{ at java.lang.reflect.Method.invoke(Method.java:498) ~[?:1.8.0_112]}}
{{ at org.apache.hadoop.hive.metastore.HiveMetaStoreClient$SynchronizedHandler.invoke(HiveMetaStoreClient.java:2956) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at com.sun.proxy.$Proxy37.listPartitionsByExpr(Unknown Source) ~[?:?]}}
{{ at org.apache.hadoop.hive.ql.metadata.Hive.getPartitionsByExpr(Hive.java:3517) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.ppr.PartitionPruner.getPartitionsFromServer(PartitionPruner.java:450) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.ppr.PartitionPruner.prune(PartitionPruner.java:227) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.ppr.PartitionPruner.prune(PartitionPruner.java:147) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.ParseContext.getPrunedPartitions(ParseContext.java:535) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.pcr.PcrOpProcFactory$FilterPCR.process(PcrOpProcFactory.java:110) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultRuleDispatcher.dispatch(DefaultRuleDispatcher.java:90) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.dispatchAndReturn(DefaultGraphWalker.java:105) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.dispatch(DefaultGraphWalker.java:89) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.walk(DefaultGraphWalker.java:158) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.lib.DefaultGraphWalker.startWalking(DefaultGraphWalker.java:120) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.pcr.PartitionConditionRemover.transform(PartitionConditionRemover.java:86) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.optimizer.Optimizer.optimize(Optimizer.java:250) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.SemanticAnalyzer.analyzeInternal(SemanticAnalyzer.java:12423) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.CalcitePlanner.analyzeInternal(CalcitePlanner.java:360) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.parse.BaseSemanticAnalyzer.analyze(BaseSemanticAnalyzer.java:289) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compile(Driver.java:664) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileInternal(Driver.java:1869) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileAndRespond(Driver.java:1816) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.Driver.compileAndRespond(Driver.java:1811) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hadoop.hive.ql.reexec.ReExecDriver.compileAndRespond(ReExecDriver.java:126) ~[hive-exec-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ at org.apache.hive.service.cli.operation.SQLOperation.prepare(SQLOperation.java:197) ~[hive-service-3.1.0.3.1.0.0-78.jar:3.1.0.3.1.0.0-78]}}
{{ ... 15 more}}



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)