Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id 73597200C6F for ; Tue, 9 May 2017 10:32:19 +0200 (CEST) Received: by cust-asf.ponee.io (Postfix) id 720D9160BB6; Tue, 9 May 2017 08:32:19 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id 8FC0B160BB3 for ; Tue, 9 May 2017 10:32:18 +0200 (CEST) Received: (qmail 76791 invoked by uid 500); 9 May 2017 08:32:17 -0000 Mailing-List: contact issues-help@spark.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Delivered-To: mailing list issues@spark.apache.org Received: (qmail 76781 invoked by uid 99); 9 May 2017 08:32:17 -0000 Received: from pnap-us-west-generic-nat.apache.org (HELO spamd1-us-west.apache.org) (209.188.14.142) by apache.org (qpsmtpd/0.29) with ESMTP; Tue, 09 May 2017 08:32:17 +0000 Received: from localhost (localhost [127.0.0.1]) by spamd1-us-west.apache.org (ASF Mail Server at spamd1-us-west.apache.org) with ESMTP id 57969C145A for ; Tue, 9 May 2017 08:32:17 +0000 (UTC) X-Virus-Scanned: Debian amavisd-new at spamd1-us-west.apache.org X-Spam-Flag: NO X-Spam-Score: -99.202 X-Spam-Level: X-Spam-Status: No, score=-99.202 tagged_above=-999 required=6.31 tests=[KAM_ASCII_DIVIDERS=0.8, RP_MATCHES_RCVD=-0.001, SPF_PASS=-0.001, USER_IN_WHITELIST=-100] autolearn=disabled Received: from mx1-lw-eu.apache.org ([10.40.0.8]) by localhost (spamd1-us-west.apache.org [10.40.0.7]) (amavisd-new, port 10024) with ESMTP id PFKAY73-l-yF for ; Tue, 9 May 2017 08:32:16 +0000 (UTC) Received: from mailrelay1-us-west.apache.org (mailrelay1-us-west.apache.org [209.188.14.139]) by mx1-lw-eu.apache.org (ASF Mail Server at mx1-lw-eu.apache.org) with ESMTP id B96535FE2F for ; Tue, 9 May 2017 08:32:15 +0000 (UTC) Received: from jira-lw-us.apache.org (unknown [207.244.88.139]) by mailrelay1-us-west.apache.org (ASF Mail Server at mailrelay1-us-west.apache.org) with ESMTP id CFFF5E0C13 for ; Tue, 9 May 2017 08:32:04 +0000 (UTC) Received: from jira-lw-us.apache.org (localhost [127.0.0.1]) by jira-lw-us.apache.org (ASF Mail Server at jira-lw-us.apache.org) with ESMTP id 33ACC21DF9 for ; Tue, 9 May 2017 08:32:04 +0000 (UTC) Date: Tue, 9 May 2017 08:32:04 +0000 (UTC) From: "Teng Yutong (JIRA)" To: issues@spark.apache.org Message-ID: In-Reply-To: References: Subject: [jira] [Commented] (SPARK-18004) DataFrame filter Predicate push-down fails for Oracle Timestamp type columns MIME-Version: 1.0 Content-Type: text/plain; charset=utf-8 Content-Transfer-Encoding: 7bit X-JIRA-FingerPrint: 30527f35849b9dde25b450d4833f0394 archived-at: Tue, 09 May 2017 08:32:19 -0000 [ https://issues.apache.org/jira/browse/SPARK-18004?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16002282#comment-16002282 ] Teng Yutong commented on SPARK-18004: ------------------------------------- a silly but doable workaround: cast TimestampType row to LongType and compare the inner representation... ``` src.filter(col("timestamp_col").cast(LongType).*(1000l) > condition.asInstanceOf[Timestamp].getTime) ``` > DataFrame filter Predicate push-down fails for Oracle Timestamp type columns > ---------------------------------------------------------------------------- > > Key: SPARK-18004 > URL: https://issues.apache.org/jira/browse/SPARK-18004 > Project: Spark > Issue Type: Bug > Components: SQL > Affects Versions: 1.6.0 > Reporter: Suhas Nalapure > Priority: Critical > > DataFrame filter Predicate push-down fails for Oracle Timestamp type columns with Exception java.sql.SQLDataException: ORA-01861: literal does not match format string: > Java source code (this code works fine for mysql & mssql databases) : > {noformat} > //DataFrame df = create a DataFrame over an Oracle table > df = df.filter(df.col("TS").lt(new java.sql.Timestamp(System.currentTimeMillis()))); > df.explain(); > df.show(); > {noformat} > Log statements with the Exception: > {noformat} > Schema: root > |-- ID: string (nullable = false) > |-- TS: timestamp (nullable = true) > |-- DEVICE_ID: string (nullable = true) > |-- REPLACEMENT: string (nullable = true) > {noformat} > {noformat} > == Physical Plan == > Filter (TS#1 < 1476861841934000) > +- Scan JDBCRelation(jdbc:oracle:thin:@10.0.0.111:1521:orcl,ORATABLE,[Lorg.apache.spark.Partition;@78c74647,{user=user, password=pwd, url=jdbc:oracle:thin:@10.0.0.111:1521:orcl, dbtable=ORATABLE, driver=oracle.jdbc.driver.OracleDriver})[ID#0,TS#1,DEVICE_ID#2,REPLACEMENT#3] PushedFilters: [LessThan(TS,2016-10-19 12:54:01.934)] > 2016-10-19 12:54:04,268 ERROR [Executor task launch worker-0] org.apache.spark.executor.Executor > Exception in task 0.0 in stage 0.0 (TID 0) > java.sql.SQLDataException: ORA-01861: literal does not match format string > at oracle.jdbc.driver.T4CTTIoer.processError(T4CTTIoer.java:461) > at oracle.jdbc.driver.T4CTTIoer.processError(T4CTTIoer.java:402) > at oracle.jdbc.driver.T4C8Oall.processError(T4C8Oall.java:1065) > at oracle.jdbc.driver.T4CTTIfun.receive(T4CTTIfun.java:681) > at oracle.jdbc.driver.T4CTTIfun.doRPC(T4CTTIfun.java:256) > at oracle.jdbc.driver.T4C8Oall.doOALL(T4C8Oall.java:577) > at oracle.jdbc.driver.T4CPreparedStatement.doOall8(T4CPreparedStatement.java:239) > at oracle.jdbc.driver.T4CPreparedStatement.doOall8(T4CPreparedStatement.java:75) > at oracle.jdbc.driver.T4CPreparedStatement.executeForDescribe(T4CPreparedStatement.java:1043) > at oracle.jdbc.driver.OracleStatement.executeMaybeDescribe(OracleStatement.java:1111) > at oracle.jdbc.driver.OracleStatement.doExecuteWithTimeout(OracleStatement.java:1353) > at oracle.jdbc.driver.OraclePreparedStatement.executeInternal(OraclePreparedStatement.java:4485) > at oracle.jdbc.driver.OraclePreparedStatement.executeQuery(OraclePreparedStatement.java:4566) > at oracle.jdbc.driver.OraclePreparedStatementWrapper.executeQuery(OraclePreparedStatementWrapper.java:5251) > at org.apache.spark.sql.execution.datasources.jdbc.JDBCRDD$$anon$1.(JDBCRDD.scala:383) > at org.apache.spark.sql.execution.datasources.jdbc.JDBCRDD.compute(JDBCRDD.scala:359) > at org.apache.spark.rdd.RDD.computeOrReadCheckpoint(RDD.scala:306) > at org.apache.spark.rdd.RDD.iterator(RDD.scala:270) > at org.apache.spark.rdd.MapPartitionsRDD.compute(MapPartitionsRDD.scala:38) > at org.apache.spark.rdd.RDD.computeOrReadCheckpoint(RDD.scala:306) > at org.apache.spark.rdd.RDD.iterator(RDD.scala:270) > at org.apache.spark.rdd.MapPartitionsRDD.compute(MapPartitionsRDD.scala:38) > at org.apache.spark.rdd.RDD.computeOrReadCheckpoint(RDD.scala:306) > at org.apache.spark.rdd.RDD.iterator(RDD.scala:270) > at org.apache.spark.scheduler.ResultTask.runTask(ResultTask.scala:66) > at org.apache.spark.scheduler.Task.run(Task.scala:89) > at org.apache.spark.executor.Executor$TaskRunner.run(Executor.scala:213) > at java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1142) > at java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:617) > at java.lang.Thread.run(Thread.java:745) > {noformat} -- This message was sent by Atlassian JIRA (v6.3.15#6346) --------------------------------------------------------------------- To unsubscribe, e-mail: issues-unsubscribe@spark.apache.org For additional commands, e-mail: issues-help@spark.apache.org