5.4. Hive Connector

Overview

The Hive connector allows querying data stored in a Hivedata warehouse. Hive is a combination of three components:

  • Data files in varying formats that are typically stored in theHadoop Distributed File System (HDFS) or in Amazon S3.
  • Metadata about how the data files are mapped to schemas and tables.This metadata is stored in a database such as MySQL and is accessedvia the Hive metastore service.
  • A query language called HiveQL. This query language is executedon a distributed computing framework such as MapReduce or Tez.
    Presto only uses the first two components: the data and the metadata.It does not use HiveQL or any part of Hive’s execution environment.

Supported File Types

The following file types are supported for the Hive connector:

  • ORC
  • Parquet
  • Avro
  • RCFile
  • SequenceFile
  • JSON
  • Text

Configuration

The Hive connector supports Apache Hadoop 2.x and derivative distributionsincluding Cloudera CDH 5 and Hortonworks Data Platform (HDP).

Create etc/catalog/hive.properties with the following contentsto mount the hive-hadoop2 connector as the hive catalog,replacing example.net:9083 with the correct host and portfor your Hive metastore Thrift service:

  1. connector.name=hive-hadoop2
  2. hive.metastore.uri=thrift://example.net:9083

Multiple Hive Clusters

You can have as many catalogs as you need, so if you have additionalHive clusters, simply add another properties file to etc/catalogwith a different name (making sure it ends in .properties). Forexample, if you name the property file sales.properties, Prestowill create a catalog named sales using the configured connector.

HDFS Configuration

For basic setups, Presto configures the HDFS client automatically anddoes not require any configuration files. In some cases, such as when usingfederated HDFS or NameNode high availability, it is necessary to specifyadditional HDFS client options in order to access your HDFS cluster. To do so,add the hive.config.resources property to reference your HDFS config files:

  1. hive.config.resources=/etc/hadoop/conf/core-site.xml,/etc/hadoop/conf/hdfs-site.xml

Only specify additional configuration files if necessary for your setup.We also recommend reducing the configuration files to have the minimumset of required properties, as additional properties may cause problems.

The configuration files must exist on all Presto nodes. If you arereferencing existing Hadoop config files, make sure to copy them toany Presto nodes that are not running Hadoop.

HDFS Username

When not using Kerberos with HDFS, Presto will access HDFS using theOS user of the Presto process. For example, if Presto is running asnobody, it will access HDFS as nobody. You can override thisusername by setting the HADOOP_USER_NAME system property in thePresto JVM Config, replacing hdfs_user with theappropriate username:

  1. -DHADOOP_USER_NAME=hdfs_user

Accessing Hadoop clusters protected with Kerberos authentication

Kerberos authentication is supported for both HDFS and the Hive metastore.However, Kerberos authentication by ticket cache is not yet supported.

The properties that apply to Hive connector security are listed in theHive Configuration Properties table. Please see theHive Security Configuration section for a more detailed discussion of thesecurity options in the Hive connector.

Hive Configuration Properties

Property Name Description Default
hive.metastore.uri The URI(s) of the Hive metastore to connect to using theThrift protocol. If multiple URIs are provided, the firstURI is used by default and the rest of the URIs arefallback metastores. This property is required.Example: thrift://192.0.2.3:9083 orthrift://192.0.2.3:9083,thrift://192.0.2.4:9083
hive.metastore.username The username Presto will use to access the Hive metastore.
hive.config.resources An optional comma-separated list of HDFSconfiguration files. These files must exist on themachines running Presto. Only specify this ifabsolutely necessary to access HDFS.Example: /etc/hdfs-site.xml
hive.storage-format The default file format used when creating new tables. RCBINARY
hive.compression-codec The compression codec to use when writing files. GZIP
hive.force-local-scheduling Force splits to be scheduled on the same node as the HadoopDataNode process serving the split data. This is useful forinstallations where Presto is collocated with everyDataNode. false
hive.respect-table-format Should new partitions be written using the existing tableformat or the default Presto format? true
hive.immutable-partitions Can new data be inserted into existing partitions? false
hive.max-partitions-per-writers Maximum number of partitions per writer. 100
hive.max-partitions-per-scan Maximum number of partitions for a single table scan. 100,000
hive.metastore.authentication.type Hive metastore authentication type.Possible values are NONE or KERBEROS. NONE
hive.metastore.service.principal The Kerberos principal of the Hive metastore service.
hive.metastore.client.principal The Kerberos principal that Presto will use when connectingto the Hive metastore service.
hive.metastore.client.keytab Hive metastore client keytab location.
hive.hdfs.authentication.type HDFS authentication type.Possible values are NONE or KERBEROS. NONE
hive.hdfs.impersonation.enabled Enable HDFS end user impersonation. false
hive.hdfs.presto.principal The Kerberos principal that Presto will use when connectingto HDFS.
hive.hdfs.presto.keytab HDFS client keytab location.
hive.security See Hive Security Configuration.
security.config-file Path of config file to use when hive.security=file.See File Based Authorization for details.
hive.non-managed-table-writes-enabled Enable writes to non-managed (external) Hive tables. false
hive.non-managed-table-creates-enabled Enable creating non-managed (external) Hive tables. true

Amazon S3 Configuration

The Hive Connector can read and write tables that are stored in S3.This is accomplished by having a table or database location thatuses an S3 prefix rather than an HDFS prefix.

Presto uses its own S3 filesystem for the URI prefixess3://, s3n:// and s3a://.

S3 Configuration Properties

Property Name Description
hive.s3.use-instance-credentials Use the EC2 metadata service to retrieve API credentials(defaults to true). This works with IAM roles in EC2.
hive.s3.aws-access-key Default AWS access key to use.
hive.s3.aws-secret-key Default AWS secret key to use.
hive.s3.endpoint The S3 storage endpoint server. This can be used toconnect to an S3-compatible storage system insteadof AWS. When using v4 signatures, it is recommended toset this to the AWS region-specific endpoint(e.g., http[s]://<bucket>.s3-<AWS-region>.amazonaws.com).
hive.s3.signer-type Specify a different signer type for S3-compatible storage.Example: S3SignerType for v2 signer type
hive.s3.path-style-access Use path-style access for all requests to the S3-compatible storage.This is for S3-compatible storage that doesn’t support virtual-hosted-style access.(defaults to false)
hive.s3.staging-directory Local staging directory for data written to S3.This defaults to the Java temporary directory specifiedby the JVM system property java.io.tmpdir.
hive.s3.pin-client-to-current-region Pin S3 requests to the same region as the EC2instance where Presto is running (defaults to false).
hive.s3.ssl.enabled Use HTTPS to communicate with the S3 API (defaults to true).
hive.s3.sse.enabled Use S3 server-side encryption (defaults to false).
hive.s3.sse.type The type of key management for S3 server-side encryption.Use S3 for S3 managed or KMS for KMS-managed keys(defaults to S3).
hive.s3.sse.kms-key-id The KMS Key ID to use for S3 server-side encryption withKMS-managed keys. If not set, the default key is used.
hive.s3.kms-key-id If set, use S3 client-side encryption and use the AWSKMS to store encryption keys and use the value ofthis property as the KMS Key ID for newly createdobjects.
hive.s3.encryption-materials-provider If set, use S3 client-side encryption and use thevalue of this property as the fully qualified name ofa Java class which implements the AWS SDK’sEncryptionMaterialsProvider interface. If theclass also implements Configurable from the HadoopAPI, the Hadoop configuration will be passed in afterthe object has been created.

S3 Credentials

If you are running Presto on Amazon EC2 using EMR or another facility,it is highly recommended that you set hive.s3.use-instance-credentialsto true and use IAM Roles for EC2 to govern access to S3. If this isthe case, your EC2 instances will need to be assigned an IAM Role whichgrants appropriate access to the data stored in the S3 bucket(s) you wishto use. This is much cleaner than setting AWS access and secret keys inthe hive.s3.aws-access-key and hive.s3.aws-secret-key settings, and alsoallows EC2 to automatically rotate credentials on a regular basis withoutany additional work on your part.

Custom S3 Credentials Provider

You can configure a custom S3 credentials provider by setting the Hadoopconfiguration property presto.s3.credentials-provider to be thefully qualified class name of a custom AWS credentials providerimplementation. This class must implement theAWSCredentialsProviderinterface and provide a two-argument constructor that takes ajava.net.URI and a Hadoop org.apache.hadoop.conf.Configurationas arguments. A custom credentials provider can be used to providetemporary credentials from STS (using STSSessionCredentialsProvider),IAM role-based credentials (using STSAssumeRoleSessionCredentialsProvider),or credentials for a specific use case (e.g., bucket/user specific credentials).This Hadoop configuration property must be set in the Hadoop configurationfiles referenced by the hive.config.resources Hive connector property.

Tuning Properties

The following tuning properties affect the behavior of the clientused by the Presto S3 filesystem when communicating with S3.Most of these parameters affect settings on the ClientConfigurationobject associated with the AmazonS3Client.

Property Name Description Default
hive.s3.max-error-retries Maximum number of error retries, set on the S3 client. 10
hive.s3.max-client-retries Maximum number of read attempts to retry. 5
hive.s3.max-backoff-time Use exponential backoff starting at 1 second up tothis maximum value when communicating with S3. 10 minutes
hive.s3.max-retry-time Maximum time to retry communicating with S3. 10 minutes
hive.s3.connect-timeout TCP connect timeout. 5 seconds
hive.s3.socket-timeout TCP socket read timeout. 5 seconds
hive.s3.max-connections Maximum number of simultaneous open connections to S3. 500
hive.s3.multipart.min-file-size Minimum file size before multi-part upload to S3 is used. 16 MB
hive.s3.multipart.min-part-size Minimum multi-part upload part size. 5 MB

S3 Data Encryption

Presto supports reading and writing encrypted data in S3 using bothserver-side encryption with S3 managed keys and client-side encryption usingeither the Amazon KMS or a software plugin to manage AES encryption keys.

With S3 server-side encryption,(called SSE-S3 in the Amazon documentation) the S3 infrastructure takes care of all encryption and decryptionwork (with the exception of SSL to the client, assuming you have hive.s3.ssl.enabled set to true).S3 also manages all the encryption keys for you. To enable this, set hive.s3.sse.enabled to true.

With S3 client-side encryption,S3 stores encrypted data and the encryption keys are managed outside of the S3 infrastructure. Data is encryptedand decrypted by Presto instead of in the S3 infrastructure. In this case, encryption keys can be managedeither by using the AWS KMS or your own key management system. To use the AWS KMS for key management, sethive.s3.kms-key-id to the UUID of a KMS key. Your AWS credentials or EC2 IAM role will need to begranted permission to use the given key as well.

To use a custom encryption key management system, set hive.s3.encryption-materials-provider to thefully qualified name of a class which implements theEncryptionMaterialsProviderinterface from the AWS Java SDK. This class will have to be accessible to the Hive Connector through theclasspath and must be able to communicate with your custom key management system. If this class also implementsthe org.apache.hadoop.conf.Configurable interface from the Hadoop Java API, then the Hadoop configurationwill be passed in after the object instance is created and before it is asked to provision or retrieve anyencryption keys.

Schema Evolution

Hive allows the partitions in a table to have a different schema than thetable. This occurs when the column types of a table are changed afterpartitions already exist (that use the original column types). The Hiveconnector supports this by allowing the same conversions as Hive:

  • varchar to and from tinyint, smallint, integer and bigint
  • real to double
  • Widening conversions for integers, such as tinyint to smallint
    Any conversion failure will result in null, which is the same behavioras Hive. For example, converting the string 'foo' to a number,or converting the string '1234' to a tinyint (which has amaximum value of 127).

Examples

The Hive connector supports querying and manipulating Hive tables and schemas(databases). While some uncommon operations will need to be performed usingHive directly, most operations can be performed using Presto.

Create a new Hive schema named web that will store tables in anS3 bucket named my-bucket:

  1. CREATE SCHEMA hive.web
  2. WITH (location = 's3://my-bucket/')

Create a new Hive table named page_views in the web schemathat is stored using the ORC file format, partitioned by date andcountry, and bucketed by user into 50 buckets (note that Hiverequires the partition columns to be the last columns in the table):

  1. CREATE TABLE hive.web.page_views (
  2. view_time timestamp,
  3. user_id bigint,
  4. page_url varchar,
  5. ds date,
  6. country varchar
  7. )
  8. WITH (
  9. format = 'ORC',
  10. partitioned_by = ARRAY['ds', 'country'],
  11. bucketed_by = ARRAY['user_id'],
  12. bucket_count = 50
  13. )

Drop a partition from the page_views table:

  1. DELETE FROM hive.web.page_views
  2. WHERE ds = DATE '2016-08-09'
  3. AND country = 'US'

Query the page_views table:

  1. SELECT * FROM hive.web.page_views

List the partitions of the page_views table:

  1. SELECT * FROM hive.web."page_views$partitions"

Create an external Hive table named request_logs that points atexisting data in S3:

  1. CREATE TABLE hive.web.request_logs (
  2. request_time timestamp,
  3. url varchar,
  4. ip varchar,
  5. user_agent varchar
  6. )
  7. WITH (
  8. format = 'TEXTFILE',
  9. external_location = 's3://my-bucket/data/logs/'
  10. )

Drop the external table request_logs. This only drops the metadatafor the table. The referenced data directory is not deleted:

  1. DROP TABLE hive.web.request_logs

Drop a schema:

  1. DROP SCHEMA hive.web

Hive Connector Limitations

DELETE is only supported if the WHERE clause matches entire partitions.

原文: https://prestodb.io/docs/current/connector/hive.html