org.apache.hadoop.hive.ql.parse.DDLSemanticAnalyzer Maven / Gradle / Ivy
Go to download
Show more of this group Show more artifacts with this name
Show all versions of hive-exec Show documentation
Show all versions of hive-exec Show documentation
Hive is a data warehouse infrastructure built on top of Hadoop see
http://wiki.apache.org/hadoop/Hive
/**
* Licensed to the Apache Software Foundation (ASF) under one
* or more contributor license agreements. See the NOTICE file
* distributed with this work for additional information
* regarding copyright ownership. The ASF licenses this file
* to you under the Apache License, Version 2.0 (the
* "License"); you may not use this file except in compliance
* with the License. You may obtain a copy of the License at
*
* http://www.apache.org/licenses/LICENSE-2.0
*
* Unless required by applicable law or agreed to in writing, software
* distributed under the License is distributed on an "AS IS" BASIS,
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
* See the License for the specific language governing permissions and
* limitations under the License.
*/
package org.apache.hadoop.hive.ql.parse;
import static org.apache.hadoop.hive.ql.parse.HiveParser.TOK_DATABASELOCATION;
import static org.apache.hadoop.hive.ql.parse.HiveParser.TOK_DATABASEPROPERTIES;
import java.io.Serializable;
import java.net.URI;
import java.net.URISyntaxException;
import java.util.ArrayList;
import java.util.HashMap;
import java.util.HashSet;
import java.util.Iterator;
import java.util.LinkedHashMap;
import java.util.LinkedList;
import java.util.List;
import java.util.Map;
import java.util.Map.Entry;
import java.util.Properties;
import java.util.Set;
import org.antlr.runtime.tree.CommonTree;
import org.antlr.runtime.tree.Tree;
import org.apache.commons.logging.Log;
import org.apache.commons.logging.LogFactory;
import org.apache.hadoop.fs.Path;
import org.apache.hadoop.hive.conf.HiveConf;
import org.apache.hadoop.hive.conf.HiveConf.ConfVars;
import org.apache.hadoop.hive.metastore.TableType;
import org.apache.hadoop.hive.metastore.Warehouse;
import org.apache.hadoop.hive.metastore.api.FieldSchema;
import org.apache.hadoop.hive.metastore.api.Index;
import org.apache.hadoop.hive.metastore.api.MetaException;
import org.apache.hadoop.hive.metastore.api.Order;
import org.apache.hadoop.hive.metastore.api.PrincipalType;
import org.apache.hadoop.hive.metastore.api.SkewedInfo;
import org.apache.hadoop.hive.ql.Driver;
import org.apache.hadoop.hive.ql.ErrorMsg;
import org.apache.hadoop.hive.ql.exec.ArchiveUtils;
import org.apache.hadoop.hive.ql.exec.FetchTask;
import org.apache.hadoop.hive.ql.exec.Task;
import org.apache.hadoop.hive.ql.exec.TaskFactory;
import org.apache.hadoop.hive.ql.exec.Utilities;
import org.apache.hadoop.hive.ql.hooks.ReadEntity;
import org.apache.hadoop.hive.ql.hooks.WriteEntity;
import org.apache.hadoop.hive.ql.index.HiveIndex;
import org.apache.hadoop.hive.ql.index.HiveIndex.IndexType;
import org.apache.hadoop.hive.ql.index.HiveIndexHandler;
import org.apache.hadoop.hive.ql.io.IgnoreKeyTextOutputFormat;
import org.apache.hadoop.hive.ql.io.RCFileInputFormat;
import org.apache.hadoop.hive.ql.io.orc.OrcBlockMergeInputFormat;
import org.apache.hadoop.hive.ql.io.orc.OrcInputFormat;
import org.apache.hadoop.hive.ql.io.orc.OrcMergeMapper;
import org.apache.hadoop.hive.ql.io.rcfile.merge.RCFileBlockMergeInputFormat;
import org.apache.hadoop.hive.ql.io.rcfile.merge.RCFileMergeMapper;
import org.apache.hadoop.hive.ql.lib.Node;
import org.apache.hadoop.hive.ql.metadata.Hive;
import org.apache.hadoop.hive.ql.metadata.HiveException;
import org.apache.hadoop.hive.ql.metadata.HiveUtils;
import org.apache.hadoop.hive.ql.metadata.Partition;
import org.apache.hadoop.hive.ql.metadata.Table;
import org.apache.hadoop.hive.ql.plan.AddPartitionDesc;
import org.apache.hadoop.hive.ql.plan.AlterDatabaseDesc;
import org.apache.hadoop.hive.ql.plan.AlterIndexDesc;
import org.apache.hadoop.hive.ql.plan.AlterIndexDesc.AlterIndexTypes;
import org.apache.hadoop.hive.ql.plan.AlterTableDesc;
import org.apache.hadoop.hive.ql.plan.AlterTableDesc.AlterTableTypes;
import org.apache.hadoop.hive.ql.plan.AlterTableSimpleDesc;
import org.apache.hadoop.hive.ql.plan.CreateDatabaseDesc;
import org.apache.hadoop.hive.ql.plan.CreateIndexDesc;
import org.apache.hadoop.hive.ql.plan.DDLWork;
import org.apache.hadoop.hive.ql.plan.DescDatabaseDesc;
import org.apache.hadoop.hive.ql.plan.DescFunctionDesc;
import org.apache.hadoop.hive.ql.plan.DescTableDesc;
import org.apache.hadoop.hive.ql.plan.DropDatabaseDesc;
import org.apache.hadoop.hive.ql.plan.DropIndexDesc;
import org.apache.hadoop.hive.ql.plan.DropTableDesc;
import org.apache.hadoop.hive.ql.plan.FetchWork;
import org.apache.hadoop.hive.ql.plan.GrantDesc;
import org.apache.hadoop.hive.ql.plan.GrantRevokeRoleDDL;
import org.apache.hadoop.hive.ql.plan.ListBucketingCtx;
import org.apache.hadoop.hive.ql.plan.LoadTableDesc;
import org.apache.hadoop.hive.ql.plan.LockTableDesc;
import org.apache.hadoop.hive.ql.plan.MoveWork;
import org.apache.hadoop.hive.ql.plan.MsckDesc;
import org.apache.hadoop.hive.ql.plan.PartitionSpec;
import org.apache.hadoop.hive.ql.plan.PlanUtils;
import org.apache.hadoop.hive.ql.plan.PrincipalDesc;
import org.apache.hadoop.hive.ql.plan.PrivilegeDesc;
import org.apache.hadoop.hive.ql.plan.PrivilegeObjectDesc;
import org.apache.hadoop.hive.ql.plan.RenamePartitionDesc;
import org.apache.hadoop.hive.ql.plan.RevokeDesc;
import org.apache.hadoop.hive.ql.plan.RoleDDLDesc;
import org.apache.hadoop.hive.ql.plan.ShowColumnsDesc;
import org.apache.hadoop.hive.ql.plan.ShowCreateTableDesc;
import org.apache.hadoop.hive.ql.plan.ShowDatabasesDesc;
import org.apache.hadoop.hive.ql.plan.ShowFunctionsDesc;
import org.apache.hadoop.hive.ql.plan.ShowGrantDesc;
import org.apache.hadoop.hive.ql.plan.ShowIndexesDesc;
import org.apache.hadoop.hive.ql.plan.ShowLocksDesc;
import org.apache.hadoop.hive.ql.plan.ShowPartitionsDesc;
import org.apache.hadoop.hive.ql.plan.ShowTableStatusDesc;
import org.apache.hadoop.hive.ql.plan.ShowTablesDesc;
import org.apache.hadoop.hive.ql.plan.ShowTblPropertiesDesc;
import org.apache.hadoop.hive.ql.plan.StatsWork;
import org.apache.hadoop.hive.ql.plan.SwitchDatabaseDesc;
import org.apache.hadoop.hive.ql.plan.TableDesc;
import org.apache.hadoop.hive.ql.plan.TruncateTableDesc;
import org.apache.hadoop.hive.ql.plan.UnlockTableDesc;
import org.apache.hadoop.hive.ql.security.authorization.Privilege;
import org.apache.hadoop.hive.ql.security.authorization.PrivilegeRegistry;
import org.apache.hadoop.hive.ql.session.SessionState;
import org.apache.hadoop.hive.serde.serdeConstants;
import org.apache.hadoop.hive.serde2.lazy.LazySimpleSerDe;
import org.apache.hadoop.mapred.InputFormat;
import org.apache.hadoop.mapred.TextInputFormat;
import org.apache.hadoop.hive.ql.plan.AlterTableAlterPartDesc;
/**
* DDLSemanticAnalyzer.
*
*/
public class DDLSemanticAnalyzer extends BaseSemanticAnalyzer {
private static final Log LOG = LogFactory.getLog(DDLSemanticAnalyzer.class);
private static final Map TokenToTypeName = new HashMap();
private final Set reservedPartitionValues;
static {
TokenToTypeName.put(HiveParser.TOK_BOOLEAN, serdeConstants.BOOLEAN_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_TINYINT, serdeConstants.TINYINT_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_SMALLINT, serdeConstants.SMALLINT_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_INT, serdeConstants.INT_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_BIGINT, serdeConstants.BIGINT_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_FLOAT, serdeConstants.FLOAT_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_DOUBLE, serdeConstants.DOUBLE_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_STRING, serdeConstants.STRING_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_BINARY, serdeConstants.BINARY_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_DATE, serdeConstants.DATE_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_DATETIME, serdeConstants.DATETIME_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_TIMESTAMP, serdeConstants.TIMESTAMP_TYPE_NAME);
TokenToTypeName.put(HiveParser.TOK_DECIMAL, serdeConstants.DECIMAL_TYPE_NAME);
}
public static String getTypeName(int token) throws SemanticException {
// date and datetime types aren't currently supported
if (token == HiveParser.TOK_DATE || token == HiveParser.TOK_DATETIME) {
throw new SemanticException(ErrorMsg.UNSUPPORTED_TYPE.getMsg());
}
return TokenToTypeName.get(token);
}
static class TablePartition {
String tableName;
HashMap partSpec = null;
public TablePartition() {
}
public TablePartition(ASTNode tblPart) throws SemanticException {
tableName = unescapeIdentifier(tblPart.getChild(0).getText());
if (tblPart.getChildCount() > 1) {
ASTNode part = (ASTNode) tblPart.getChild(1);
if (part.getToken().getType() == HiveParser.TOK_PARTSPEC) {
this.partSpec = DDLSemanticAnalyzer.getPartSpec(part);
}
}
}
}
public DDLSemanticAnalyzer(HiveConf conf) throws SemanticException {
super(conf);
reservedPartitionValues = new HashSet();
// Partition can't have this name
reservedPartitionValues.add(HiveConf.getVar(conf, ConfVars.DEFAULTPARTITIONNAME));
reservedPartitionValues.add(HiveConf.getVar(conf, ConfVars.DEFAULT_ZOOKEEPER_PARTITION_NAME));
// Partition value can't end in this suffix
reservedPartitionValues.add(HiveConf.getVar(conf, ConfVars.METASTORE_INT_ORIGINAL));
reservedPartitionValues.add(HiveConf.getVar(conf, ConfVars.METASTORE_INT_ARCHIVED));
reservedPartitionValues.add(HiveConf.getVar(conf, ConfVars.METASTORE_INT_EXTRACTED));
}
@Override
public void analyzeInternal(ASTNode ast) throws SemanticException {
switch (ast.getToken().getType()) {
case HiveParser.TOK_ALTERTABLE_PARTITION: {
ASTNode tablePart = (ASTNode) ast.getChild(0);
TablePartition tblPart = new TablePartition(tablePart);
String tableName = tblPart.tableName;
HashMap partSpec = tblPart.partSpec;
ast = (ASTNode) ast.getChild(1);
if (ast.getToken().getType() == HiveParser.TOK_ALTERTABLE_FILEFORMAT) {
analyzeAlterTableFileFormat(ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_ALTERTABLE_ALTERPARTS_PROTECTMODE) {
analyzeAlterTableProtectMode(ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_ALTERTABLE_LOCATION) {
analyzeAlterTableLocation(ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_ALTERTABLE_ALTERPARTS_MERGEFILES) {
analyzeAlterTablePartMergeFiles(tablePart, ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_ALTERTABLE_SERIALIZER) {
analyzeAlterTableSerde(ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_ALTERTABLE_SERDEPROPERTIES) {
analyzeAlterTableSerdeProps(ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_ALTERTABLE_RENAMEPART) {
analyzeAlterTableRenamePart(ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_ALTERTBLPART_SKEWED_LOCATION) {
analyzeAlterTableSkewedLocation(ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_TABLEBUCKETS) {
analyzeAlterTableBucketNum(ast, tableName, partSpec);
} else if (ast.getToken().getType() == HiveParser.TOK_ALTERTABLE_CLUSTER_SORT) {
analyzeAlterTableClusterSort(ast, tableName, partSpec);
}
break;
}
case HiveParser.TOK_DROPTABLE:
analyzeDropTable(ast, false);
break;
case HiveParser.TOK_TRUNCATETABLE:
analyzeTruncateTable(ast);
break;
case HiveParser.TOK_CREATEINDEX:
analyzeCreateIndex(ast);
break;
case HiveParser.TOK_DROPINDEX:
analyzeDropIndex(ast);
break;
case HiveParser.TOK_DESCTABLE:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeDescribeTable(ast);
break;
case HiveParser.TOK_SHOWDATABASES:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowDatabases(ast);
break;
case HiveParser.TOK_SHOWTABLES:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowTables(ast);
break;
case HiveParser.TOK_SHOWCOLUMNS:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowColumns(ast);
break;
case HiveParser.TOK_SHOW_TABLESTATUS:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowTableStatus(ast);
break;
case HiveParser.TOK_SHOW_TBLPROPERTIES:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowTableProperties(ast);
break;
case HiveParser.TOK_SHOWFUNCTIONS:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowFunctions(ast);
break;
case HiveParser.TOK_SHOWLOCKS:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowLocks(ast);
break;
case HiveParser.TOK_DESCFUNCTION:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeDescFunction(ast);
break;
case HiveParser.TOK_DESCDATABASE:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeDescDatabase(ast);
break;
case HiveParser.TOK_MSCK:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeMetastoreCheck(ast);
break;
case HiveParser.TOK_DROPVIEW:
analyzeDropTable(ast, true);
break;
case HiveParser.TOK_ALTERVIEW_PROPERTIES:
analyzeAlterTableProps(ast, true, false);
break;
case HiveParser.TOK_DROPVIEW_PROPERTIES:
analyzeAlterTableProps(ast, true, true);
break;
case HiveParser.TOK_ALTERVIEW_ADDPARTS:
// for ALTER VIEW ADD PARTITION, we wrapped the ADD to discriminate
// view from table; unwrap it now
analyzeAlterTableAddParts((ASTNode) ast.getChild(0), true);
break;
case HiveParser.TOK_ALTERVIEW_DROPPARTS:
// for ALTER VIEW DROP PARTITION, we wrapped the DROP to discriminate
// view from table; unwrap it now
analyzeAlterTableDropParts((ASTNode) ast.getChild(0), true);
break;
case HiveParser.TOK_ALTERVIEW_RENAME:
// for ALTER VIEW RENAME, we wrapped the RENAME to discriminate
// view from table; unwrap it now
analyzeAlterTableRename(((ASTNode) ast.getChild(0)), true);
break;
case HiveParser.TOK_ALTERTABLE_RENAME:
analyzeAlterTableRename(ast, false);
break;
case HiveParser.TOK_ALTERTABLE_TOUCH:
analyzeAlterTableTouch(ast);
break;
case HiveParser.TOK_ALTERTABLE_ARCHIVE:
analyzeAlterTableArchive(ast, false);
break;
case HiveParser.TOK_ALTERTABLE_UNARCHIVE:
analyzeAlterTableArchive(ast, true);
break;
case HiveParser.TOK_ALTERTABLE_ADDCOLS:
analyzeAlterTableModifyCols(ast, AlterTableTypes.ADDCOLS);
break;
case HiveParser.TOK_ALTERTABLE_REPLACECOLS:
analyzeAlterTableModifyCols(ast, AlterTableTypes.REPLACECOLS);
break;
case HiveParser.TOK_ALTERTABLE_RENAMECOL:
analyzeAlterTableRenameCol(ast);
break;
case HiveParser.TOK_ALTERTABLE_ADDPARTS:
analyzeAlterTableAddParts(ast, false);
break;
case HiveParser.TOK_ALTERTABLE_DROPPARTS:
analyzeAlterTableDropParts(ast, false);
break;
case HiveParser.TOK_ALTERTABLE_ALTERPARTS:
analyzeAlterTableAlterParts(ast);
break;
case HiveParser.TOK_ALTERTABLE_PROPERTIES:
analyzeAlterTableProps(ast, false, false);
break;
case HiveParser.TOK_DROPTABLE_PROPERTIES:
analyzeAlterTableProps(ast, false, true);
break;
case HiveParser.TOK_ALTERINDEX_REBUILD:
analyzeAlterIndexRebuild(ast);
break;
case HiveParser.TOK_ALTERINDEX_PROPERTIES:
analyzeAlterIndexProps(ast);
break;
case HiveParser.TOK_SHOWPARTITIONS:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowPartitions(ast);
break;
case HiveParser.TOK_SHOW_CREATETABLE:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowCreateTable(ast);
break;
case HiveParser.TOK_SHOWINDEXES:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowIndexes(ast);
break;
case HiveParser.TOK_LOCKTABLE:
analyzeLockTable(ast);
break;
case HiveParser.TOK_UNLOCKTABLE:
analyzeUnlockTable(ast);
break;
case HiveParser.TOK_CREATEDATABASE:
analyzeCreateDatabase(ast);
break;
case HiveParser.TOK_DROPDATABASE:
analyzeDropDatabase(ast);
break;
case HiveParser.TOK_SWITCHDATABASE:
analyzeSwitchDatabase(ast);
break;
case HiveParser.TOK_ALTERDATABASE_PROPERTIES:
analyzeAlterDatabase(ast);
break;
case HiveParser.TOK_CREATEROLE:
analyzeCreateRole(ast);
break;
case HiveParser.TOK_DROPROLE:
analyzeDropRole(ast);
break;
case HiveParser.TOK_SHOW_ROLE_GRANT:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowRoleGrant(ast);
break;
case HiveParser.TOK_GRANT_ROLE:
analyzeGrantRevokeRole(true, ast);
break;
case HiveParser.TOK_REVOKE_ROLE:
analyzeGrantRevokeRole(false, ast);
break;
case HiveParser.TOK_GRANT:
analyzeGrant(ast);
break;
case HiveParser.TOK_SHOW_GRANT:
ctx.setResFile(new Path(ctx.getLocalTmpFileURI()));
analyzeShowGrant(ast);
break;
case HiveParser.TOK_REVOKE:
analyzeRevoke(ast);
break;
case HiveParser.TOK_ALTERTABLE_SKEWED:
analyzeAltertableSkewedby(ast);
break;
default:
throw new SemanticException("Unsupported command.");
}
}
private void analyzeGrantRevokeRole(boolean grant, ASTNode ast) {
List principalDesc = analyzePrincipalListDef(
(ASTNode) ast.getChild(0));
List roles = new ArrayList();
for (int i = 1; i < ast.getChildCount(); i++) {
roles.add(unescapeIdentifier(ast.getChild(i).getText()));
}
String roleOwnerName = "";
if (SessionState.get() != null
&& SessionState.get().getAuthenticator() != null) {
roleOwnerName = SessionState.get().getAuthenticator().getUserName();
}
GrantRevokeRoleDDL grantRevokeRoleDDL = new GrantRevokeRoleDDL(grant,
roles, principalDesc, roleOwnerName, PrincipalType.USER, true);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
grantRevokeRoleDDL), conf));
}
private void analyzeShowGrant(ASTNode ast) throws SemanticException {
PrivilegeObjectDesc privHiveObj = null;
ASTNode principal = (ASTNode) ast.getChild(0);
PrincipalType type = PrincipalType.USER;
switch (principal.getType()) {
case HiveParser.TOK_USER:
type = PrincipalType.USER;
break;
case HiveParser.TOK_GROUP:
type = PrincipalType.GROUP;
break;
case HiveParser.TOK_ROLE:
type = PrincipalType.ROLE;
break;
}
String principalName = unescapeIdentifier(principal.getChild(0).getText());
PrincipalDesc principalDesc = new PrincipalDesc(principalName, type);
List cols = null;
if (ast.getChildCount() > 1) {
ASTNode child = (ASTNode) ast.getChild(1);
if (child.getToken().getType() == HiveParser.TOK_PRIV_OBJECT_COL) {
privHiveObj = new PrivilegeObjectDesc();
privHiveObj.setObject(unescapeIdentifier(child.getChild(0).getText()));
if (child.getChildCount() > 1) {
for (int i = 1; i < child.getChildCount(); i++) {
ASTNode grandChild = (ASTNode) child.getChild(i);
if (grandChild.getToken().getType() == HiveParser.TOK_PARTSPEC) {
privHiveObj.setPartSpec(DDLSemanticAnalyzer.getPartSpec(grandChild));
} else if (grandChild.getToken().getType() == HiveParser.TOK_TABCOLNAME) {
cols = getColumnNames((ASTNode) grandChild);
} else {
privHiveObj.setTable(child.getChild(i) != null);
}
}
}
}
}
if (privHiveObj == null && cols != null) {
throw new SemanticException(
"For user-level privileges, column sets should be null. columns="
+ cols.toString());
}
ShowGrantDesc showGrant = new ShowGrantDesc(ctx.getResFile().toString(),
principalDesc, privHiveObj, cols);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
showGrant), conf));
}
private void analyzeGrant(ASTNode ast) throws SemanticException {
List privilegeDesc = analyzePrivilegeListDef(
(ASTNode) ast.getChild(0));
List principalDesc = analyzePrincipalListDef(
(ASTNode) ast.getChild(1));
boolean grantOption = false;
PrivilegeObjectDesc privilegeObj = null;
if (ast.getChildCount() > 2) {
for (int i = 2; i < ast.getChildCount(); i++) {
ASTNode astChild = (ASTNode) ast.getChild(i);
if (astChild.getType() == HiveParser.TOK_GRANT_WITH_OPTION) {
grantOption = true;
} else if (astChild.getType() == HiveParser.TOK_PRIV_OBJECT) {
privilegeObj = analyzePrivilegeObject(astChild, getOutputs());
}
}
}
String userName = null;
if (SessionState.get() != null
&& SessionState.get().getAuthenticator() != null) {
userName = SessionState.get().getAuthenticator().getUserName();
}
GrantDesc grantDesc = new GrantDesc(privilegeObj, privilegeDesc,
principalDesc, userName, PrincipalType.USER, grantOption);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
grantDesc), conf));
}
private void analyzeRevoke(ASTNode ast) throws SemanticException {
List privilegeDesc = analyzePrivilegeListDef(
(ASTNode) ast.getChild(0));
List principalDesc = analyzePrincipalListDef(
(ASTNode) ast.getChild(1));
PrivilegeObjectDesc hiveObj = null;
if (ast.getChildCount() > 2) {
ASTNode astChild = (ASTNode) ast.getChild(2);
hiveObj = analyzePrivilegeObject(astChild, getOutputs());
}
RevokeDesc revokeDesc = new RevokeDesc(privilegeDesc, principalDesc, hiveObj);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
revokeDesc), conf));
}
private PrivilegeObjectDesc analyzePrivilegeObject(ASTNode ast,
HashSet outputs)
throws SemanticException {
PrivilegeObjectDesc subject = new PrivilegeObjectDesc();
subject.setObject(unescapeIdentifier(ast.getChild(0).getText()));
if (ast.getChildCount() > 1) {
for (int i = 0; i < ast.getChildCount(); i++) {
ASTNode astChild = (ASTNode) ast.getChild(i);
if (astChild.getToken().getType() == HiveParser.TOK_PARTSPEC) {
subject.setPartSpec(DDLSemanticAnalyzer.getPartSpec(astChild));
} else {
subject.setTable(ast.getChild(0) != null);
}
}
}
if (subject.getTable()) {
Table tbl = getTable(subject.getObject(), true);
if (subject.getPartSpec() != null) {
Partition part = getPartition(tbl, subject.getPartSpec(), true);
outputs.add(new WriteEntity(part));
} else {
outputs.add(new WriteEntity(tbl));
}
}
return subject;
}
private List analyzePrincipalListDef(ASTNode node) {
List principalList = new ArrayList();
for (int i = 0; i < node.getChildCount(); i++) {
ASTNode child = (ASTNode) node.getChild(i);
PrincipalType type = null;
switch (child.getType()) {
case HiveParser.TOK_USER:
type = PrincipalType.USER;
break;
case HiveParser.TOK_GROUP:
type = PrincipalType.GROUP;
break;
case HiveParser.TOK_ROLE:
type = PrincipalType.ROLE;
break;
}
String principalName = unescapeIdentifier(child.getChild(0).getText());
PrincipalDesc principalDesc = new PrincipalDesc(principalName, type);
principalList.add(principalDesc);
}
return principalList;
}
private List analyzePrivilegeListDef(ASTNode node)
throws SemanticException {
List ret = new ArrayList();
for (int i = 0; i < node.getChildCount(); i++) {
ASTNode privilegeDef = (ASTNode) node.getChild(i);
ASTNode privilegeType = (ASTNode) privilegeDef.getChild(0);
Privilege privObj = PrivilegeRegistry.getPrivilege(privilegeType.getType());
if (privObj == null) {
throw new SemanticException("undefined privilege " + privilegeType.getType());
}
List cols = null;
if (privilegeDef.getChildCount() > 1) {
cols = getColumnNames((ASTNode) privilegeDef.getChild(1));
}
PrivilegeDesc privilegeDesc = new PrivilegeDesc(privObj, cols);
ret.add(privilegeDesc);
}
return ret;
}
private void analyzeCreateRole(ASTNode ast) {
String roleName = unescapeIdentifier(ast.getChild(0).getText());
RoleDDLDesc createRoleDesc = new RoleDDLDesc(roleName,
RoleDDLDesc.RoleOperation.CREATE_ROLE);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
createRoleDesc), conf));
}
private void analyzeDropRole(ASTNode ast) {
String roleName = unescapeIdentifier(ast.getChild(0).getText());
RoleDDLDesc createRoleDesc = new RoleDDLDesc(roleName,
RoleDDLDesc.RoleOperation.DROP_ROLE);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
createRoleDesc), conf));
}
private void analyzeShowRoleGrant(ASTNode ast) {
ASTNode child = (ASTNode) ast.getChild(0);
PrincipalType principalType = PrincipalType.USER;
switch (child.getType()) {
case HiveParser.TOK_USER:
principalType = PrincipalType.USER;
break;
case HiveParser.TOK_GROUP:
principalType = PrincipalType.GROUP;
break;
case HiveParser.TOK_ROLE:
principalType = PrincipalType.ROLE;
break;
}
String principalName = unescapeIdentifier(child.getChild(0).getText());
RoleDDLDesc createRoleDesc = new RoleDDLDesc(principalName, principalType,
RoleDDLDesc.RoleOperation.SHOW_ROLE_GRANT, null);
createRoleDesc.setResFile(ctx.getResFile().toString());
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
createRoleDesc), conf));
}
private void analyzeAlterDatabase(ASTNode ast) throws SemanticException {
String dbName = unescapeIdentifier(ast.getChild(0).getText());
Map dbProps = null;
for (int i = 1; i < ast.getChildCount(); i++) {
ASTNode childNode = (ASTNode) ast.getChild(i);
switch (childNode.getToken().getType()) {
case HiveParser.TOK_DATABASEPROPERTIES:
dbProps = DDLSemanticAnalyzer.getProps((ASTNode) childNode.getChild(0));
break;
default:
throw new SemanticException("Unrecognized token in CREATE DATABASE statement");
}
}
// currently alter database command can only change properties
AlterDatabaseDesc alterDesc = new AlterDatabaseDesc(dbName, null, null, false);
alterDesc.setDatabaseProperties(dbProps);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(), alterDesc),
conf));
}
private void analyzeCreateDatabase(ASTNode ast) throws SemanticException {
String dbName = unescapeIdentifier(ast.getChild(0).getText());
boolean ifNotExists = false;
String dbComment = null;
String dbLocation = null;
Map dbProps = null;
for (int i = 1; i < ast.getChildCount(); i++) {
ASTNode childNode = (ASTNode) ast.getChild(i);
switch (childNode.getToken().getType()) {
case HiveParser.TOK_IFNOTEXISTS:
ifNotExists = true;
break;
case HiveParser.TOK_DATABASECOMMENT:
dbComment = unescapeSQLString(childNode.getChild(0).getText());
break;
case TOK_DATABASEPROPERTIES:
dbProps = DDLSemanticAnalyzer.getProps((ASTNode) childNode.getChild(0));
break;
case TOK_DATABASELOCATION:
dbLocation = unescapeSQLString(childNode.getChild(0).getText());
break;
default:
throw new SemanticException("Unrecognized token in CREATE DATABASE statement");
}
}
CreateDatabaseDesc createDatabaseDesc =
new CreateDatabaseDesc(dbName, dbComment, dbLocation, ifNotExists);
if (dbProps != null) {
createDatabaseDesc.setDatabaseProperties(dbProps);
}
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
createDatabaseDesc), conf));
}
private void analyzeDropDatabase(ASTNode ast) throws SemanticException {
String dbName = unescapeIdentifier(ast.getChild(0).getText());
boolean ifExists = false;
boolean ifCascade = false;
if (null != ast.getFirstChildWithType(HiveParser.TOK_IFEXISTS)) {
ifExists = true;
}
if (null != ast.getFirstChildWithType(HiveParser.TOK_CASCADE)) {
ifCascade = true;
}
DropDatabaseDesc dropDatabaseDesc = new DropDatabaseDesc(dbName, ifExists, ifCascade);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(), dropDatabaseDesc), conf));
}
private void analyzeSwitchDatabase(ASTNode ast) {
String dbName = unescapeIdentifier(ast.getChild(0).getText());
SwitchDatabaseDesc switchDatabaseDesc = new SwitchDatabaseDesc(dbName);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
switchDatabaseDesc), conf));
}
private void analyzeDropTable(ASTNode ast, boolean expectView)
throws SemanticException {
String tableName = getUnescapedName((ASTNode) ast.getChild(0));
boolean ifExists = (ast.getFirstChildWithType(HiveParser.TOK_IFEXISTS) != null);
// we want to signal an error if the table/view doesn't exist and we're
// configured not to fail silently
boolean throwException =
!ifExists && !HiveConf.getBoolVar(conf, ConfVars.DROPIGNORESNONEXISTENT);
Table tab = getTable(tableName, throwException);
if (tab != null) {
inputs.add(new ReadEntity(tab));
outputs.add(new WriteEntity(tab));
}
DropTableDesc dropTblDesc = new DropTableDesc(
tableName, expectView, ifExists, true);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
dropTblDesc), conf));
}
private void analyzeTruncateTable(ASTNode ast) throws SemanticException {
ASTNode root = (ASTNode) ast.getChild(0); // TOK_TABLE_PARTITION
String tableName = getUnescapedName((ASTNode) root.getChild(0));
Table table = getTable(tableName, true);
if (table.getTableType() != TableType.MANAGED_TABLE) {
throw new SemanticException(ErrorMsg.TRUNCATE_FOR_NON_MANAGED_TABLE.format(tableName));
}
if (table.isNonNative()) {
throw new SemanticException(ErrorMsg.TRUNCATE_FOR_NON_NATIVE_TABLE.format(tableName)); //TODO
}
if (!table.isPartitioned() && root.getChildCount() > 1) {
throw new SemanticException(ErrorMsg.PARTSPEC_FOR_NON_PARTITIONED_TABLE.format(tableName));
}
Map partSpec = getPartSpec((ASTNode) root.getChild(1));
if (partSpec == null) {
if (!table.isPartitioned()) {
outputs.add(new WriteEntity(table));
} else {
for (Partition partition : getPartitions(table, null, false)) {
outputs.add(new WriteEntity(partition));
}
}
} else {
if (isFullSpec(table, partSpec)) {
Partition partition = getPartition(table, partSpec, true);
outputs.add(new WriteEntity(partition));
} else {
for (Partition partition : getPartitions(table, partSpec, false)) {
outputs.add(new WriteEntity(partition));
}
}
}
TruncateTableDesc truncateTblDesc = new TruncateTableDesc(tableName, partSpec);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(), truncateTblDesc), conf));
}
private boolean isFullSpec(Table table, Map partSpec) {
for (FieldSchema partCol : table.getPartCols()) {
if (partSpec.get(partCol.getName()) == null) {
return false;
}
}
return true;
}
private void analyzeCreateIndex(ASTNode ast) throws SemanticException {
String indexName = unescapeIdentifier(ast.getChild(0).getText());
String typeName = unescapeSQLString(ast.getChild(1).getText());
String tableName = getUnescapedName((ASTNode) ast.getChild(2));
List indexedCols = getColumnNames((ASTNode) ast.getChild(3));
IndexType indexType = HiveIndex.getIndexType(typeName);
if (indexType != null) {
typeName = indexType.getHandlerClsName();
} else {
try {
Class.forName(typeName);
} catch (Exception e) {
throw new SemanticException("class name provided for index handler not found.", e);
}
}
String indexTableName = null;
boolean deferredRebuild = false;
String location = null;
Map tblProps = null;
Map idxProps = null;
String indexComment = null;
RowFormatParams rowFormatParams = new RowFormatParams();
StorageFormat storageFormat = new StorageFormat();
AnalyzeCreateCommonVars shared = new AnalyzeCreateCommonVars();
for (int idx = 4; idx < ast.getChildCount(); idx++) {
ASTNode child = (ASTNode) ast.getChild(idx);
if (storageFormat.fillStorageFormat(child, shared)) {
continue;
}
switch (child.getToken().getType()) {
case HiveParser.TOK_TABLEROWFORMAT:
rowFormatParams.analyzeRowFormat(shared, child);
break;
case HiveParser.TOK_CREATEINDEX_INDEXTBLNAME:
ASTNode ch = (ASTNode) child.getChild(0);
indexTableName = getUnescapedName((ASTNode) ch);
break;
case HiveParser.TOK_DEFERRED_REBUILDINDEX:
deferredRebuild = true;
break;
case HiveParser.TOK_TABLELOCATION:
location = unescapeSQLString(child.getChild(0).getText());
break;
case HiveParser.TOK_TABLEPROPERTIES:
tblProps = DDLSemanticAnalyzer.getProps((ASTNode) child.getChild(0));
break;
case HiveParser.TOK_INDEXPROPERTIES:
idxProps = DDLSemanticAnalyzer.getProps((ASTNode) child.getChild(0));
break;
case HiveParser.TOK_TABLESERIALIZER:
child = (ASTNode) child.getChild(0);
shared.serde = unescapeSQLString(child.getChild(0).getText());
if (child.getChildCount() == 2) {
readProps((ASTNode) (child.getChild(1).getChild(0)),
shared.serdeProps);
}
break;
case HiveParser.TOK_INDEXCOMMENT:
child = (ASTNode) child.getChild(0);
indexComment = unescapeSQLString(child.getText());
}
}
storageFormat.fillDefaultStorageFormat(shared);
CreateIndexDesc crtIndexDesc = new CreateIndexDesc(tableName, indexName,
indexedCols, indexTableName, deferredRebuild, storageFormat.inputFormat,
storageFormat.outputFormat,
storageFormat.storageHandler, typeName, location, idxProps, tblProps,
shared.serde, shared.serdeProps, rowFormatParams.collItemDelim,
rowFormatParams.fieldDelim, rowFormatParams.fieldEscape,
rowFormatParams.lineDelim, rowFormatParams.mapKeyDelim, indexComment);
Task> createIndex =
TaskFactory.get(new DDLWork(getInputs(), getOutputs(), crtIndexDesc), conf);
rootTasks.add(createIndex);
}
private void analyzeDropIndex(ASTNode ast) throws SemanticException {
String indexName = unescapeIdentifier(ast.getChild(0).getText());
String tableName = getUnescapedName((ASTNode) ast.getChild(1));
boolean ifExists = (ast.getFirstChildWithType(HiveParser.TOK_IFEXISTS) != null);
// we want to signal an error if the index doesn't exist and we're
// configured not to ignore this
boolean throwException =
!ifExists && !HiveConf.getBoolVar(conf, ConfVars.DROPIGNORESNONEXISTENT);
if (throwException) {
try {
Index idx = db.getIndex(tableName, indexName);
} catch (HiveException e) {
throw new SemanticException(ErrorMsg.INVALID_INDEX.getMsg(indexName));
}
}
DropIndexDesc dropIdxDesc = new DropIndexDesc(indexName, tableName);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
dropIdxDesc), conf));
}
private void analyzeAlterIndexRebuild(ASTNode ast) throws SemanticException {
String baseTableName = unescapeIdentifier(ast.getChild(0).getText());
String indexName = unescapeIdentifier(ast.getChild(1).getText());
HashMap partSpec = null;
Tree part = ast.getChild(2);
if (part != null) {
partSpec = extractPartitionSpecs(part);
}
List> indexBuilder = getIndexBuilderMapRed(baseTableName, indexName, partSpec);
rootTasks.addAll(indexBuilder);
// Handle updating index timestamps
AlterIndexDesc alterIdxDesc = new AlterIndexDesc(AlterIndexTypes.UPDATETIMESTAMP);
alterIdxDesc.setIndexName(indexName);
alterIdxDesc.setBaseTableName(baseTableName);
alterIdxDesc.setDbName(db.getCurrentDatabase());
alterIdxDesc.setSpec(partSpec);
Task> tsTask = TaskFactory.get(new DDLWork(alterIdxDesc), conf);
for (Task> t : indexBuilder) {
t.addDependentTask(tsTask);
}
}
private void analyzeAlterIndexProps(ASTNode ast)
throws SemanticException {
String baseTableName = getUnescapedName((ASTNode) ast.getChild(0));
String indexName = unescapeIdentifier(ast.getChild(1).getText());
HashMap mapProp = getProps((ASTNode) (ast.getChild(2))
.getChild(0));
AlterIndexDesc alterIdxDesc =
new AlterIndexDesc(AlterIndexTypes.ADDPROPS);
alterIdxDesc.setProps(mapProp);
alterIdxDesc.setIndexName(indexName);
alterIdxDesc.setBaseTableName(baseTableName);
alterIdxDesc.setDbName(db.getCurrentDatabase());
rootTasks.add(TaskFactory.get(new DDLWork(alterIdxDesc), conf));
}
private List> getIndexBuilderMapRed(String baseTableName, String indexName,
HashMap partSpec) throws SemanticException {
try {
String dbName = db.getCurrentDatabase();
Index index = db.getIndex(dbName, baseTableName, indexName);
Table indexTbl = getTable(index.getIndexTableName());
String baseTblName = index.getOrigTableName();
Table baseTbl = getTable(baseTblName);
String handlerCls = index.getIndexHandlerClass();
HiveIndexHandler handler = HiveUtils.getIndexHandler(conf, handlerCls);
List indexTblPartitions = null;
List baseTblPartitions = null;
if (indexTbl != null) {
indexTblPartitions = new ArrayList();
baseTblPartitions = preparePartitions(baseTbl, partSpec,
indexTbl, db, indexTblPartitions);
}
List> ret = handler.generateIndexBuildTaskList(baseTbl,
index, indexTblPartitions, baseTblPartitions, indexTbl, getInputs(), getOutputs());
return ret;
} catch (Exception e) {
throw new SemanticException(e);
}
}
private List preparePartitions(
org.apache.hadoop.hive.ql.metadata.Table baseTbl,
HashMap partSpec,
org.apache.hadoop.hive.ql.metadata.Table indexTbl, Hive db,
List indexTblPartitions)
throws HiveException, MetaException {
List baseTblPartitions = new ArrayList();
if (partSpec != null) {
// if partspec is specified, then only producing index for that
// partition
Partition part = db.getPartition(baseTbl, partSpec, false);
if (part == null) {
throw new HiveException("Partition "
+ Warehouse.makePartName(partSpec, false)
+ " does not exist in table "
+ baseTbl.getTableName());
}
baseTblPartitions.add(part);
Partition indexPart = db.getPartition(indexTbl, partSpec, false);
if (indexPart == null) {
indexPart = db.createPartition(indexTbl, partSpec);
}
indexTblPartitions.add(indexPart);
} else if (baseTbl.isPartitioned()) {
// if no partition is specified, create indexes for all partitions one
// by one.
baseTblPartitions = db.getPartitions(baseTbl);
for (Partition basePart : baseTblPartitions) {
HashMap pSpec = basePart.getSpec();
Partition indexPart = db.getPartition(indexTbl, pSpec, false);
if (indexPart == null) {
indexPart = db.createPartition(indexTbl, pSpec);
}
indexTblPartitions.add(indexPart);
}
}
return baseTblPartitions;
}
private void validateAlterTableType(Table tbl, AlterTableTypes op) throws SemanticException {
validateAlterTableType(tbl, op, false);
}
private void validateAlterTableType(Table tbl, AlterTableTypes op, boolean expectView)
throws SemanticException {
if (tbl.isView()) {
if (!expectView) {
throw new SemanticException(ErrorMsg.ALTER_COMMAND_FOR_VIEWS.getMsg());
}
switch (op) {
case ADDPARTITION:
case DROPPARTITION:
case RENAMEPARTITION:
case ADDPROPS:
case DROPPROPS:
case RENAME:
// allow this form
break;
default:
throw new SemanticException(ErrorMsg.ALTER_VIEW_DISALLOWED_OP.getMsg(op.toString()));
}
} else {
if (expectView) {
throw new SemanticException(ErrorMsg.ALTER_COMMAND_FOR_TABLES.getMsg());
}
}
if (tbl.isNonNative()) {
throw new SemanticException(ErrorMsg.ALTER_TABLE_NON_NATIVE.getMsg(tbl.getTableName()));
}
}
private void analyzeAlterTableProps(ASTNode ast, boolean expectView, boolean isUnset)
throws SemanticException {
String tableName = getUnescapedName((ASTNode) ast.getChild(0));
HashMap mapProp = getProps((ASTNode) (ast.getChild(1))
.getChild(0));
AlterTableDesc alterTblDesc = null;
if (isUnset == true) {
alterTblDesc = new AlterTableDesc(AlterTableTypes.DROPPROPS, expectView);
if (ast.getChild(2) != null) {
alterTblDesc.setDropIfExists(true);
}
} else {
alterTblDesc = new AlterTableDesc(AlterTableTypes.ADDPROPS, expectView);
}
alterTblDesc.setProps(mapProp);
alterTblDesc.setOldName(tableName);
addInputsOutputsAlterTable(tableName, null, alterTblDesc);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
alterTblDesc), conf));
}
private void analyzeAlterTableSerdeProps(ASTNode ast, String tableName,
HashMap partSpec)
throws SemanticException {
HashMap mapProp = getProps((ASTNode) (ast.getChild(0))
.getChild(0));
AlterTableDesc alterTblDesc = new AlterTableDesc(
AlterTableTypes.ADDSERDEPROPS);
alterTblDesc.setProps(mapProp);
alterTblDesc.setOldName(tableName);
alterTblDesc.setPartSpec(partSpec);
addInputsOutputsAlterTable(tableName, partSpec, alterTblDesc);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
alterTblDesc), conf));
}
private void analyzeAlterTableSerde(ASTNode ast, String tableName,
HashMap partSpec)
throws SemanticException {
String serdeName = unescapeSQLString(ast.getChild(0).getText());
AlterTableDesc alterTblDesc = new AlterTableDesc(AlterTableTypes.ADDSERDE);
if (ast.getChildCount() > 1) {
HashMap mapProp = getProps((ASTNode) (ast.getChild(1))
.getChild(0));
alterTblDesc.setProps(mapProp);
}
alterTblDesc.setOldName(tableName);
alterTblDesc.setSerdeName(serdeName);
alterTblDesc.setPartSpec(partSpec);
addInputsOutputsAlterTable(tableName, partSpec, alterTblDesc);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
alterTblDesc), conf));
}
private void analyzeAlterTableFileFormat(ASTNode ast, String tableName,
HashMap partSpec)
throws SemanticException {
String inputFormat = null;
String outputFormat = null;
String storageHandler = null;
String serde = null;
ASTNode child = (ASTNode) ast.getChild(0);
switch (child.getToken().getType()) {
case HiveParser.TOK_TABLEFILEFORMAT:
inputFormat = unescapeSQLString(((ASTNode) child.getChild(0)).getToken()
.getText());
outputFormat = unescapeSQLString(((ASTNode) child.getChild(1)).getToken()
.getText());
try {
Class.forName(inputFormat);
Class.forName(outputFormat);
} catch (ClassNotFoundException e) {
throw new SemanticException(e);
}
break;
case HiveParser.TOK_STORAGEHANDLER:
storageHandler =
unescapeSQLString(((ASTNode) child.getChild(1)).getToken().getText());
try {
Class.forName(storageHandler);
} catch (ClassNotFoundException e) {
throw new SemanticException(e);
}
break;
case HiveParser.TOK_TBLSEQUENCEFILE:
inputFormat = SEQUENCEFILE_INPUT;
outputFormat = SEQUENCEFILE_OUTPUT;
break;
case HiveParser.TOK_TBLTEXTFILE:
inputFormat = TEXTFILE_INPUT;
outputFormat = TEXTFILE_OUTPUT;
break;
case HiveParser.TOK_TBLRCFILE:
inputFormat = RCFILE_INPUT;
outputFormat = RCFILE_OUTPUT;
serde = COLUMNAR_SERDE;
break;
case HiveParser.TOK_TBLORCFILE:
inputFormat = ORCFILE_INPUT;
outputFormat = ORCFILE_OUTPUT;
serde = ORCFILE_SERDE;
break;
case HiveParser.TOK_FILEFORMAT_GENERIC:
handleGenericFileFormat(child);
break;
}
AlterTableDesc alterTblDesc = new AlterTableDesc(tableName, inputFormat,
outputFormat, serde, storageHandler, partSpec);
addInputsOutputsAlterTable(tableName, partSpec, alterTblDesc);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
alterTblDesc), conf));
}
private void addInputsOutputsAlterTable(String tableName, Map partSpec)
throws SemanticException {
addInputsOutputsAlterTable(tableName, partSpec, null);
}
private void addInputsOutputsAlterTable(String tableName, Map partSpec,
AlterTableDesc desc) throws SemanticException {
Table tab = getTable(tableName, true);
if (partSpec == null || partSpec.isEmpty()) {
inputs.add(new ReadEntity(tab));
outputs.add(new WriteEntity(tab));
}
else {
inputs.add(new ReadEntity(tab));
if (desc == null || desc.getOp() != AlterTableDesc.AlterTableTypes.ALTERPROTECTMODE) {
Partition part = getPartition(tab, partSpec, true);
outputs.add(new WriteEntity(part));
}
else {
for (Partition part : getPartitions(tab, partSpec, true)) {
outputs.add(new WriteEntity(part));
}
}
}
if (desc != null) {
validateAlterTableType(tab, desc.getOp(), desc.getExpectView());
// validate Unset Non Existed Table Properties
if (desc.getOp() == AlterTableDesc.AlterTableTypes.DROPPROPS &&
desc.getIsDropIfExists() == false) {
Iterator keyItr = desc.getProps().keySet().iterator();
while (keyItr.hasNext()) {
String currKey = keyItr.next();
if (tab.getTTable().getParameters().containsKey(currKey) == false) {
String errorMsg =
"The following property " + currKey +
" does not exist in " + tab.getTableName();
throw new SemanticException(
ErrorMsg.ALTER_TBL_UNSET_NON_EXIST_PROPERTY.getMsg(errorMsg));
}
}
}
}
}
private void analyzeAlterTableLocation(ASTNode ast, String tableName,
HashMap partSpec) throws SemanticException {
String newLocation = unescapeSQLString(ast.getChild(0).getText());
AlterTableDesc alterTblDesc = new AlterTableDesc(tableName, newLocation, partSpec);
addInputsOutputsAlterTable(tableName, partSpec, alterTblDesc);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
alterTblDesc), conf));
}
private void analyzeAlterTableProtectMode(ASTNode ast, String tableName,
HashMap partSpec)
throws SemanticException {
AlterTableDesc alterTblDesc =
new AlterTableDesc(AlterTableTypes.ALTERPROTECTMODE);
alterTblDesc.setOldName(tableName);
alterTblDesc.setPartSpec(partSpec);
ASTNode child = (ASTNode) ast.getChild(0);
switch (child.getToken().getType()) {
case HiveParser.TOK_ENABLE:
alterTblDesc.setProtectModeEnable(true);
break;
case HiveParser.TOK_DISABLE:
alterTblDesc.setProtectModeEnable(false);
break;
default:
throw new SemanticException(
"Set Protect mode Syntax parsing error.");
}
ASTNode grandChild = (ASTNode) child.getChild(0);
switch (grandChild.getToken().getType()) {
case HiveParser.TOK_OFFLINE:
alterTblDesc.setProtectModeType(AlterTableDesc.ProtectModeType.OFFLINE);
break;
case HiveParser.TOK_NO_DROP:
if (grandChild.getChildCount() > 0) {
alterTblDesc.setProtectModeType(AlterTableDesc.ProtectModeType.NO_DROP_CASCADE);
}
else {
alterTblDesc.setProtectModeType(AlterTableDesc.ProtectModeType.NO_DROP);
}
break;
case HiveParser.TOK_READONLY:
throw new SemanticException(
"Potect mode READONLY is not implemented");
default:
throw new SemanticException(
"Only protect mode NO_DROP or OFFLINE supported");
}
addInputsOutputsAlterTable(tableName, partSpec, alterTblDesc);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
alterTblDesc), conf));
}
private void analyzeAlterTablePartMergeFiles(ASTNode tablePartAST, ASTNode ast,
String tableName, HashMap partSpec)
throws SemanticException {
AlterTablePartMergeFilesDesc mergeDesc = new AlterTablePartMergeFilesDesc(
tableName, partSpec);
List inputDir = new ArrayList();
Path oldTblPartLoc = null;
Path newTblPartLoc = null;
Table tblObj = null;
ListBucketingCtx lbCtx = null;
try {
tblObj = getTable(tableName);
List bucketCols = null;
Class extends InputFormat> inputFormatClass = null;
boolean isArchived = false;
boolean checkIndex = HiveConf.getBoolVar(conf,
HiveConf.ConfVars.HIVE_CONCATENATE_CHECK_INDEX);
if (checkIndex) {
List indexes = db.getIndexes(tblObj.getDbName(), tableName,
Short.MAX_VALUE);
if (indexes != null && indexes.size() > 0) {
throw new SemanticException("can not do merge because source table "
+ tableName + " is indexed.");
}
}
if (tblObj.isPartitioned()) {
if (partSpec == null) {
throw new SemanticException("source table " + tableName
+ " is partitioned but no partition desc found.");
} else {
Partition part = getPartition(tblObj, partSpec, false);
if (part == null) {
throw new SemanticException("source table " + tableName
+ " is partitioned but partition not found.");
}
bucketCols = part.getBucketCols();
inputFormatClass = part.getInputFormatClass();
isArchived = ArchiveUtils.isArchived(part);
Path tabPath = tblObj.getPath();
Path partPath = part.getPartitionPath();
// if the table is in a different dfs than the partition,
// replace the partition's dfs with the table's dfs.
newTblPartLoc = new Path(tabPath.toUri().getScheme(), tabPath.toUri()
.getAuthority(), partPath.toUri().getPath());
oldTblPartLoc = partPath;
lbCtx = constructListBucketingCtx(part.getSkewedColNames(), part.getSkewedColValues(),
part.getSkewedColValueLocationMaps(), part.isStoredAsSubDirectories(), conf);
}
} else {
inputFormatClass = tblObj.getInputFormatClass();
bucketCols = tblObj.getBucketCols();
// input and output are the same
oldTblPartLoc = tblObj.getPath();
newTblPartLoc = tblObj.getPath();
lbCtx = constructListBucketingCtx(tblObj.getSkewedColNames(), tblObj.getSkewedColValues(),
tblObj.getSkewedColValueLocationMaps(), tblObj.isStoredAsSubDirectories(), conf);
}
// throw a HiveException for non-rcfile.
if (inputFormatClass.equals(RCFileInputFormat.class)) {
mergeDesc.setMergeMapperClass(RCFileMergeMapper.class);
mergeDesc.setMergeInputFormatClass(RCFileBlockMergeInputFormat.class);
} else if (inputFormatClass.equals(OrcInputFormat.class)) {
mergeDesc.setMergeMapperClass(OrcMergeMapper.class);
mergeDesc.setMergeInputFormatClass(OrcBlockMergeInputFormat.class);
} else {
throw new SemanticException(
"Only RCFileFormat and OrcFileFormat are supportted right now.");
}
// throw a HiveException if the table/partition is bucketized
if (bucketCols != null && bucketCols.size() > 0) {
throw new SemanticException(
"Merge can not perform on bucketized partition/table.");
}
// throw a HiveException if the table/partition is archived
if (isArchived) {
throw new SemanticException(
"Merge can not perform on archived partitions.");
}
inputDir.add(oldTblPartLoc.toString());
mergeDesc.setInputDir(inputDir);
mergeDesc.setLbCtx(lbCtx);
addInputsOutputsAlterTable(tableName, partSpec);
DDLWork ddlWork = new DDLWork(getInputs(), getOutputs(), mergeDesc);
ddlWork.setNeedLock(true);
Task extends Serializable> mergeTask = TaskFactory.get(ddlWork, conf);
TableDesc tblDesc = Utilities.getTableDesc(tblObj);
String queryTmpdir = ctx.getExternalTmpFileURI(newTblPartLoc.toUri());
mergeDesc.setOutputDir(queryTmpdir);
LoadTableDesc ltd = new LoadTableDesc(queryTmpdir, queryTmpdir, tblDesc,
partSpec == null ? new HashMap() : partSpec);
ltd.setLbCtx(lbCtx);
Task moveTsk = TaskFactory.get(new MoveWork(null, null, ltd, null, false),
conf);
mergeTask.addDependentTask(moveTsk);
if (conf.getBoolVar(HiveConf.ConfVars.HIVESTATSAUTOGATHER)) {
StatsWork statDesc;
if (oldTblPartLoc.equals(newTblPartLoc)) {
// If we're merging to the same location, we can avoid some metastore calls
tableSpec tablepart = new tableSpec(this.db, conf, tablePartAST);
statDesc = new StatsWork(tablepart);
} else {
statDesc = new StatsWork(ltd);
}
statDesc.setNoStatsAggregator(true);
statDesc.setStatsReliable(conf.getBoolVar(HiveConf.ConfVars.HIVE_STATS_RELIABLE));
Task extends Serializable> statTask = TaskFactory.get(statDesc, conf);
moveTsk.addDependentTask(statTask);
}
rootTasks.add(mergeTask);
} catch (Exception e) {
throw new SemanticException(e);
}
}
private void analyzeAlterTableClusterSort(ASTNode ast, String tableName,
HashMap partSpec) throws SemanticException {
addInputsOutputsAlterTable(tableName, partSpec);
AlterTableDesc alterTblDesc;
switch (ast.getChild(0).getType()) {
case HiveParser.TOK_NOT_CLUSTERED:
alterTblDesc = new AlterTableDesc(tableName, -1, new ArrayList(),
new ArrayList(), partSpec);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(), alterTblDesc), conf));
break;
case HiveParser.TOK_NOT_SORTED:
alterTblDesc = new AlterTableDesc(tableName, true, partSpec);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(), alterTblDesc), conf));
break;
case HiveParser.TOK_TABLEBUCKETS:
ASTNode buckets = (ASTNode) ast.getChild(0);
List bucketCols = getColumnNames((ASTNode) buckets.getChild(0));
List sortCols = new ArrayList();
int numBuckets = -1;
if (buckets.getChildCount() == 2) {
numBuckets = (Integer.valueOf(buckets.getChild(1).getText())).intValue();
} else {
sortCols = getColumnNamesOrder((ASTNode) buckets.getChild(1));
numBuckets = (Integer.valueOf(buckets.getChild(2).getText())).intValue();
}
if (numBuckets <= 0) {
throw new SemanticException(ErrorMsg.INVALID_BUCKET_NUMBER.getMsg());
}
alterTblDesc = new AlterTableDesc(tableName, numBuckets,
bucketCols, sortCols, partSpec);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
alterTblDesc), conf));
break;
}
}
static HashMap getProps(ASTNode prop) {
HashMap mapProp = new HashMap();
readProps(prop, mapProp);
return mapProp;
}
/**
* Utility class to resolve QualifiedName
*/
static class QualifiedNameUtil {
// delimiter to check DOT delimited qualified names
static String delimiter = "\\.";
/**
* Get the fully qualified name in the ast. e.g. the ast of the form ^(DOT
* ^(DOT a b) c) will generate a name of the form a.b.c
*
* @param ast
* The AST from which the qualified name has to be extracted
* @return String
*/
static public String getFullyQualifiedName(ASTNode ast) {
if (ast.getChildCount() == 0) {
return ast.getText();
} else if (ast.getChildCount() == 2) {
return getFullyQualifiedName((ASTNode) ast.getChild(0)) + "."
+ getFullyQualifiedName((ASTNode) ast.getChild(1));
} else if (ast.getChildCount() == 3) {
return getFullyQualifiedName((ASTNode) ast.getChild(0)) + "."
+ getFullyQualifiedName((ASTNode) ast.getChild(1)) + "."
+ getFullyQualifiedName((ASTNode) ast.getChild(2));
} else {
return null;
}
}
// assume the first component of DOT delimited name is tableName
// get the attemptTableName
static public String getAttemptTableName(Hive db, String qualifiedName, boolean isColumn) {
// check whether the name starts with table
// DESCRIBE table
// DESCRIBE table.column
// DECRIBE table column
String tableName = qualifiedName.substring(0,
qualifiedName.indexOf('.') == -1 ?
qualifiedName.length() : qualifiedName.indexOf('.'));
try {
Table tab = db.getTable(tableName);
if (tab != null) {
if (isColumn) {
// if attempt to get columnPath
// return the whole qualifiedName(table.column or table)
return qualifiedName;
} else {
// if attempt to get tableName
// return table
return tableName;
}
}
} catch (HiveException e) {
// assume the first DOT delimited component is tableName
// OK if it is not
// do nothing when having exception
return null;
}
return null;
}
// get Database Name
static public String getDBName(Hive db, ASTNode ast) {
String dbName = null;
String fullyQualifiedName = getFullyQualifiedName(ast);
// if database.table or database.table.column or table.column
// first try the first component of the DOT separated name
if (ast.getChildCount() >= 2) {
dbName = fullyQualifiedName.substring(0,
fullyQualifiedName.indexOf('.') == -1 ?
fullyQualifiedName.length() :
fullyQualifiedName.indexOf('.'));
try {
// if the database name is not valid
// it is table.column
// return null as dbName
if (!db.databaseExists(dbName)) {
return null;
}
} catch (HiveException e) {
return null;
}
} else {
// in other cases, return null
// database is not validated if null
return null;
}
return dbName;
}
// get Table Name
static public String getTableName(Hive db, ASTNode ast)
throws SemanticException {
String tableName = null;
String fullyQualifiedName = getFullyQualifiedName(ast);
// assume the first component of DOT delimited name is tableName
String attemptTableName = getAttemptTableName(db, fullyQualifiedName, false);
if (attemptTableName != null) {
return attemptTableName;
}
// if the name does not start with table
// it should start with database
// DESCRIBE database.table
// DESCRIBE database.table column
if (fullyQualifiedName.split(delimiter).length == 3) {
// if DESCRIBE database.table.column
// invalid syntax exception
if (ast.getChildCount() == 2) {
throw new SemanticException(ErrorMsg.INVALID_TABLE_OR_COLUMN.getMsg(fullyQualifiedName));
} else {
// if DESCRIBE database.table column
// return database.table as tableName
tableName = fullyQualifiedName.substring(0,
fullyQualifiedName.lastIndexOf('.'));
}
} else if (fullyQualifiedName.split(delimiter).length == 2) {
// if DESCRIBE database.table
// return database.table as tableName
tableName = fullyQualifiedName;
} else {
// if fullyQualifiedName only have one component
// it is an invalid table
throw new SemanticException(ErrorMsg.INVALID_TABLE.getMsg(fullyQualifiedName));
}
return tableName;
}
// get column path
static public String getColPath(
Hive db,
ASTNode parentAst,
ASTNode ast,
String tableName,
Map partSpec) {
// if parent has two children
// it could be DESCRIBE table key
// or DESCRIBE table partition
if (parentAst.getChildCount() == 2 && partSpec == null) {
// if partitionSpec is null
// it is DESCRIBE table key
// return table as columnPath
return getFullyQualifiedName(parentAst);
}
// assume the first component of DOT delimited name is tableName
String attemptTableName = getAttemptTableName(db, tableName, true);
if (attemptTableName != null) {
return attemptTableName;
}
// if the name does not start with table
// it should start with database
// DESCRIBE database.table
// DESCRIBE database.table column
if (tableName.split(delimiter).length == 3) {
// if DESCRIBE database.table column
// return table.column as column path
return tableName.substring(
tableName.indexOf(".") + 1, tableName.length());
}
// in other cases, column path is the same as tableName
return tableName;
}
// get partition metadata
static public Map getPartitionSpec(Hive db, ASTNode ast, String tableName)
throws SemanticException {
// if ast has two children
// it could be DESCRIBE table key
// or DESCRIBE table partition
// check whether it is DESCRIBE table partition
if (ast.getChildCount() == 2) {
ASTNode partNode = (ASTNode) ast.getChild(1);
HashMap partSpec = null;
try {
partSpec = getPartSpec(partNode);
} catch (SemanticException e) {
// get exception in resolving partition
// it could be DESCRIBE table key
// return null
// continue processing for DESCRIBE table key
return null;
}
Table tab = null;
try {
tab = db.getTable(tableName);
} catch (HiveException e) {
// if table not valid
// throw semantic exception
throw new SemanticException(ErrorMsg.INVALID_TABLE.getMsg(tableName), e);
}
if (partSpec != null) {
Partition part = null;
try {
part = db.getPartition(tab, partSpec, false);
} catch (HiveException e) {
// if get exception in finding partition
// it could be DESCRIBE table key
// return null
// continue processing for DESCRIBE table key
return null;
}
// if partition is not found
// it is DESCRIBE table partition
// invalid partition exception
if (part == null) {
throw new SemanticException(ErrorMsg.INVALID_PARTITION.getMsg(partSpec.toString()));
}
// it is DESCRIBE table partition
// return partition metadata
return partSpec;
}
}
return null;
}
}
/**
* Create a FetchTask for a given table and thrift ddl schema.
*
* @param tablename
* tablename
* @param schema
* thrift ddl
*/
private FetchTask createFetchTask(String schema) {
Properties prop = new Properties();
prop.setProperty(serdeConstants.SERIALIZATION_FORMAT, "9");
prop.setProperty(serdeConstants.SERIALIZATION_NULL_FORMAT, " ");
String[] colTypes = schema.split("#");
prop.setProperty("columns", colTypes[0]);
prop.setProperty("columns.types", colTypes[1]);
FetchWork fetch = new FetchWork(ctx.getResFile().toString(), new TableDesc(
LazySimpleSerDe.class, TextInputFormat.class,
IgnoreKeyTextOutputFormat.class, prop), -1);
fetch.setSerializationNullFormat(" ");
return (FetchTask) TaskFactory.get(fetch, conf);
}
private void validateDatabase(String databaseName) throws SemanticException {
try {
if (!db.databaseExists(databaseName)) {
throw new SemanticException(ErrorMsg.DATABASE_NOT_EXISTS.getMsg(databaseName));
}
} catch (HiveException e) {
throw new SemanticException(ErrorMsg.DATABASE_NOT_EXISTS.getMsg(databaseName), e);
}
}
private void validateTable(String tableName, Map partSpec)
throws SemanticException {
Table tab = getTable(tableName);
if (partSpec != null) {
getPartition(tab, partSpec, true);
}
}
private void analyzeDescribeTable(ASTNode ast) throws SemanticException {
ASTNode tableTypeExpr = (ASTNode) ast.getChild(0);
String qualifiedName =
QualifiedNameUtil.getFullyQualifiedName((ASTNode) tableTypeExpr.getChild(0));
String tableName =
QualifiedNameUtil.getTableName(db, (ASTNode)(tableTypeExpr.getChild(0)));
String dbName =
QualifiedNameUtil.getDBName(db, (ASTNode)(tableTypeExpr.getChild(0)));
Map partSpec =
QualifiedNameUtil.getPartitionSpec(db, tableTypeExpr, tableName);
String colPath = QualifiedNameUtil.getColPath(
db, tableTypeExpr, (ASTNode) tableTypeExpr.getChild(0), qualifiedName, partSpec);
// if database is not the one currently using
// validate database
if (dbName != null) {
validateDatabase(dbName);
}
if (partSpec != null) {
validateTable(tableName, partSpec);
}
DescTableDesc descTblDesc = new DescTableDesc(
ctx.getResFile(), tableName, partSpec, colPath);
if (ast.getChildCount() == 2) {
int descOptions = ast.getChild(1).getType();
descTblDesc.setFormatted(descOptions == HiveParser.KW_FORMATTED);
descTblDesc.setExt(descOptions == HiveParser.KW_EXTENDED);
descTblDesc.setPretty(descOptions == HiveParser.KW_PRETTY);
}
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(),
descTblDesc), conf));
setFetchTask(createFetchTask(DescTableDesc.getSchema()));
LOG.info("analyzeDescribeTable done");
}
/**
* Describe database.
*
* @param ast
* @throws SemanticException
*/
private void analyzeDescDatabase(ASTNode ast) throws SemanticException {
boolean isExtended;
String dbName;
if (ast.getChildCount() == 1) {
dbName = stripQuotes(ast.getChild(0).getText());
isExtended = false;
} else if (ast.getChildCount() == 2) {
dbName = stripQuotes(ast.getChild(0).getText());
isExtended = true;
} else {
throw new SemanticException("Unexpected Tokens at DESCRIBE DATABASE");
}
DescDatabaseDesc descDbDesc = new DescDatabaseDesc(ctx.getResFile(),
dbName, isExtended);
rootTasks.add(TaskFactory.get(new DDLWork(getInputs(), getOutputs(), descDbDesc), conf));
setFetchTask(createFetchTask(descDbDesc.getSchema()));
}
private static HashMap getPartSpec(ASTNode partspec)
throws SemanticException {
if (partspec == null) {
return null;
}
HashMap partSpec = new LinkedHashMap();
for (int i = 0; i < partspec.getChildCount(); ++i) {
ASTNode partspec_val = (ASTNode) partspec.getChild(i);
String key = partspec_val.getChild(0).getText();
String val = null;
if (partspec_val.getChildCount() > 1) {
val = stripQuotes(partspec_val.getChild(1).getText());
}
partSpec.put(key.toLowerCase(), val);
}
return partSpec;
}
private void analyzeShowPartitions(ASTNode ast) throws SemanticException {
ShowPartitionsDesc showPartsDesc;
String tableName = getUnescapedName((ASTNode) ast.getChild(0));
List
© 2015 - 2025 Weber Informatics LLC | Privacy Policy