Commit 35670609 by Neeru Gupta Committed by Dave Kantor

ATLAS-737 Add DSL support for Sum, Max, Min and count operations with and…

ATLAS-737 Add DSL support for Sum, Max, Min and count operations with and without group by (guptaneeru via dkantor)
parent 6fb7b82a
......@@ -11,7 +11,7 @@ The grammar for the DSL is below.
<verbatim>
queryWithPath: query ~ opt(WITHPATH)
query: querySrc ~ opt(loopExpression) ~ opt(selectClause) ~ opt(orderby) ~ opt(limitOffset)
query: querySrc ~ opt(loopExpression) ~ opt(groupByExpr) ~ opt(selectClause) ~ opt(orderby) ~ opt(limitOffset)
querySrc: rep1sep(singleQrySrc, opt(COMMA))
......@@ -22,6 +22,8 @@ singleQrySrc = FROM ~ fromSrc ~ opt(WHERE) ~ opt(expr ^? notIdExpression) |
fromSrc: identifier ~ AS ~ alias | identifier
groupByExpr = GROUPBY ~ (LPAREN ~> rep1sep(selectExpression, COMMA) <~ RPAREN)
orderby: ORDERBY ~ expr ~ opt (sortOrder)
limitOffset: LIMIT ~ lmt ~ opt (offset)
......@@ -34,6 +36,14 @@ loopExpression: LOOP ~ (LPAREN ~> query <~ RPAREN) ~ opt(intConstant <~ TIMES) ~
selectClause: SELECT ~ rep1sep(selectExpression, COMMA)
countClause = COUNT ~ LPAREN ~ RPAREN
maxClause = MAX ~ (LPAREN ~> expr <~ RPAREN)
minClause = MIN ~ (LPAREN ~> expr <~ RPAREN)
sumClause = SUM ~ (LPAREN ~> expr <~ RPAREN)
selectExpression: expr ~ opt(AS ~> alias)
expr: compE ~ opt(rep(exprRight))
......@@ -44,7 +54,7 @@ compE:
arithE ~ (LT | LTE | EQ | NEQ | GT | GTE) ~ arithE |
arithE ~ (ISA | IS) ~ ident |
arithE ~ HAS ~ ident |
arithE
arithE | countClause | maxClause | minClause | sumClause
arithE: multiE ~ opt(rep(arithERight))
......@@ -87,6 +97,7 @@ Language Notes:
* The _!WithPath_ clause can be used with transitive closure queries to retrieve the Path that
connects the two related Entities. (We also provide a higher level interface for Closure Queries
see scaladoc for 'org.apache.atlas.query.ClosureQuery')
* GROUPBY is optional. Group by can be specified with aggregate methods like max, min, sum and count. When group by is specified aggregated results are returned based on the method specified in select clause. Select expression is mandatory with group by expression.
* ORDERBY is optional. When order by clause is specified, case insensitive sorting is done based on the column specified.
For sorting in descending order specify 'DESC' after order by clause. If no order by is specified, then no default sorting is applied.
* LIMIT is optional. It limits the maximum number of objects to be fetched starting from specified optional offset. If no offset is specified count starts from beginning.
......@@ -116,6 +127,10 @@ DSL queries:
* Column where Column isa PII
* Table where name="sales_fact", columns
* Table where name="sales_fact", columns as column select column.name, column.dataType, column.comment
* DB groupby(owner) select owner, count()
* DB groupby(owner) select owner, max(name)
* DB groupby(owner) select owner, min(name)
* from Person select count() as 'count', max(Person.age) as 'max', min(Person.age)
* `Log Data`
---++ Full-text Search
......
......@@ -9,6 +9,7 @@ ATLAS-1060 Add composite indexes for exact match performance improvements for al
ATLAS-1127 Modify creation and modification timestamps to Date instead of Long(sumasai)
ALL CHANGES:
ATLAS-737 Add DSL support for Sum, Max, Min and count operations with and without group by (guptaneeru via dkantor)
ATLAS-1305 Fix potential NPEs in instance conversion code (sumasai)
ATLAS-1349 Reduce excessive exception logging (apoorvnaik via svimal2106)
ATLAS-1343 CTAS query is not captured by Atlas with Hive2 (svimal2106)
......
......@@ -22,6 +22,7 @@ import java.util.ArrayList;
import java.util.List;
import org.apache.atlas.AtlasException;
import org.apache.atlas.groovy.CastExpression;
import org.apache.atlas.groovy.ClosureExpression;
import org.apache.atlas.groovy.ComparisonExpression;
import org.apache.atlas.groovy.ComparisonOperatorExpression;
......@@ -34,6 +35,7 @@ import org.apache.atlas.groovy.LiteralExpression;
import org.apache.atlas.groovy.LogicalExpression;
import org.apache.atlas.groovy.RangeExpression;
import org.apache.atlas.groovy.TernaryOperatorExpression;
import org.apache.atlas.groovy.TypeCoersionExpression;
import org.apache.atlas.groovy.ComparisonExpression.ComparisonOperator;
import org.apache.atlas.groovy.LogicalExpression.LogicalOperator;
import org.apache.atlas.query.GraphPersistenceStrategies;
......@@ -53,8 +55,9 @@ public class Gremlin2ExpressionFactory extends GremlinExpressionFactory {
private static final String PATH_FIELD = "path";
private static final String ENABLE_PATH_METHOD = "enablePath";
private static final String BACK_METHOD = "back";
private static final String VERTEX_LIST_CLASS = "List<Vertex>";
private static final String VERTEX_ARRAY_CLASS = "Vertex[]";
private static final String LAST_METHOD = "last";
@Override
public GroovyExpression generateLogicalExpression(GroovyExpression parent, String operator, List<GroovyExpression> operands) {
return new FunctionCallExpression(parent, operator, operands);
......@@ -63,9 +66,12 @@ public class Gremlin2ExpressionFactory extends GremlinExpressionFactory {
@Override
public GroovyExpression generateBackReferenceExpression(GroovyExpression parent, boolean inSelect, String alias) {
if (inSelect) {
if (inSelect && parent == null) {
return getFieldInSelect();
}
else if (inSelect && parent != null) {
return parent;
}
else {
return new FunctionCallExpression(parent, BACK_METHOD, new LiteralExpression(alias));
}
......@@ -208,16 +214,44 @@ public class Gremlin2ExpressionFactory extends GremlinExpressionFactory {
return new FunctionCallExpression(parent, ORDER_METHOD, new ClosureExpression(comparisonFunction));
}
@Override
public GroovyExpression getAnonymousTraversalExpression() {
return new FunctionCallExpression("_");
}
@Override
public GroovyExpression generateGroupByExpression(GroovyExpression parent, GroovyExpression groupByExpression,
GroovyExpression aggregationFunction) {
GroovyExpression groupByClosureExpr = new ClosureExpression(groupByExpression);
GroovyExpression itClosure = new ClosureExpression(getItVariable());
GroovyExpression result = new FunctionCallExpression(parent, "groupBy", groupByClosureExpr, itClosure);
result = new FunctionCallExpression(result, "cap");
result = new FunctionCallExpression(result, "next");
result = new FunctionCallExpression(result, "values");
result = new FunctionCallExpression(result, "toList");
GroovyExpression mapValuesClosure = new ClosureExpression(getItVariable());
GroovyExpression aggregrationFunctionClosure = new ClosureExpression(aggregationFunction);
result = new FunctionCallExpression(result, "collect", aggregrationFunctionClosure);
return result;
}
@Override
public GroovyExpression getFieldInSelect() {
return getItVariable();
}
@Override
public GroovyExpression getGroupBySelectFieldParent() {
GroovyExpression itExpr = getItVariable();
return new FunctionCallExpression(itExpr, LAST_METHOD);
}
//assumes cast already performed
@Override
public GroovyExpression generateCountExpression(GroovyExpression itExpr) {
GroovyExpression collectionExpr = new CastExpression(itExpr,"Collection");
return new FunctionCallExpression(itExpr, "size");
}
}
......@@ -279,7 +279,9 @@ public class Gremlin3ExpressionFactory extends GremlinExpressionFactory {
public GroovyExpression generateOrderByExpression(GroovyExpression parent, List<GroovyExpression> translatedOrderBy,
boolean isAscending) {
GroovyExpression orderByClause = translatedOrderBy.get(0);
GroovyExpression orderByExpr = translatedOrderBy.get(0);
GroovyExpression orderByClosure = new ClosureExpression(orderByExpr);
GroovyExpression orderByClause = new TypeCoersionExpression(orderByClosure, FUNCTION_CLASS);
GroovyExpression aExpr = new IdentifierExpression("a");
GroovyExpression bExpr = new IdentifierExpression("b");
......@@ -322,4 +324,32 @@ public class Gremlin3ExpressionFactory extends GremlinExpressionFactory {
return new FunctionCallExpression(expr2, LAST_METHOD);
}
@Override
public GroovyExpression generateGroupByExpression(GroovyExpression parent, GroovyExpression groupByExpression,
GroovyExpression aggregationFunction) {
GroovyExpression result = new FunctionCallExpression(parent, "group");
GroovyExpression groupByClosureExpr = new TypeCoersionExpression(new ClosureExpression(groupByExpression), "Function");
result = new FunctionCallExpression(result, "by", groupByClosureExpr);
result = new FunctionCallExpression(result, "toList");
GroovyExpression mapValuesClosure = new ClosureExpression(new FunctionCallExpression(new CastExpression(getItVariable(), "Map"), "values"));
result = new FunctionCallExpression(result, "collect", mapValuesClosure);
//when we call Map.values(), we end up with an extra list around the result. We remove this by calling toList().get(0). This
//leaves us with a list of lists containing the vertices that match each group. We then apply the aggregation functions
//specified in the select list to each of these inner lists.
result = new FunctionCallExpression(result ,"toList");
result = new FunctionCallExpression(result, "get", new LiteralExpression(0));
GroovyExpression aggregrationFunctionClosure = new ClosureExpression(aggregationFunction);
result = new FunctionCallExpression(result, "collect", aggregrationFunctionClosure);
return result;
}
public GroovyExpression getGroupBySelectFieldParent() {
return null;
}
}
......@@ -23,6 +23,7 @@ import java.util.List;
import org.apache.atlas.AtlasException;
import org.apache.atlas.groovy.ArithmeticExpression;
import org.apache.atlas.groovy.CastExpression;
import org.apache.atlas.groovy.ClosureExpression;
import org.apache.atlas.groovy.FieldExpression;
import org.apache.atlas.groovy.FunctionCallExpression;
......@@ -214,7 +215,6 @@ public abstract class GremlinExpressionFactory {
*/
protected abstract GroovyExpression initialExpression(GraphPersistenceStrategies s, GroovyExpression varExpr);
/**
* Generates an expression that tests whether the vertex represented by the 'toTest'
* expression represents an instance of the specified type, checking both the type
......@@ -385,13 +385,12 @@ public abstract class GremlinExpressionFactory {
return new ArithmeticExpression(left, op, right);
}
public abstract GroovyExpression generateGroupByExpression(GroovyExpression parent, GroovyExpression groupByExpression, GroovyExpression aggregationFunction);
protected GroovyExpression getItVariable() {
return new IdentifierExpression(IT_VARIABLE);
}
protected GroovyExpression getAllVerticesExpr() {
GroovyExpression gExpr = getGraph();
return new FunctionCallExpression(gExpr, V_METHOD);
......@@ -401,9 +400,40 @@ public abstract class GremlinExpressionFactory {
return new IdentifierExpression(G_VARIABLE);
}
protected GroovyExpression getCurrentObjectExpression() {
return new FieldExpression(getItVariable(), OBJECT_FIELD);
}
//assumes cast already performed
public GroovyExpression generateCountExpression(GroovyExpression itExpr) {
GroovyExpression collectionExpr = new CastExpression(itExpr,"Collection");
return new FunctionCallExpression(collectionExpr, "size");
}
public GroovyExpression generateMinExpression(GroovyExpression itExpr, GroovyExpression mapFunction) {
return getAggregrationExpression(itExpr, mapFunction, "min");
}
public GroovyExpression generateMaxExpression(GroovyExpression itExpr, GroovyExpression mapFunction) {
return getAggregrationExpression(itExpr, mapFunction, "max");
}
public GroovyExpression generateSumExpression(GroovyExpression itExpr, GroovyExpression mapFunction) {
return getAggregrationExpression(itExpr, mapFunction, "sum");
}
private GroovyExpression getAggregrationExpression(GroovyExpression itExpr,
GroovyExpression mapFunction, String functionName) {
GroovyExpression collectionExpr = new CastExpression(itExpr,
"Collection");
ClosureExpression collectFunction = new ClosureExpression(mapFunction);
GroovyExpression transformedList = new FunctionCallExpression(
collectionExpr, "collect", collectFunction);
return new FunctionCallExpression(transformedList, functionName);
}
public GroovyExpression getClosureArgumentValue() {
return getItVariable();
}
public abstract GroovyExpression getGroupBySelectFieldParent();
}
\ No newline at end of file
......@@ -62,7 +62,7 @@ object Expressions {
trait Expression {
self: Product =>
def isAggregator = false
def children: Seq[Expression]
/**
......@@ -331,6 +331,12 @@ object Expressions {
def limit(lmt: Literal[Integer], offset : Literal[Integer]) = new LimitExpression(this, lmt, offset)
def order(odr: Expression, asc: Boolean) = new OrderExpression(this, odr, asc)
def max(maxClause: Expression) = new MaxExpression(maxClause)
def min(minClause: Expression) = new MinExpression(minClause)
def groupBy(groupBy: SelectExpression, selectExpr: SelectExpression) = new GroupByExpression(this, groupBy, selectExpr)
}
trait BinaryNode {
......@@ -393,7 +399,7 @@ object Expressions {
case class UnresolvedFieldExpression(child: Expression, fieldName: String) extends Expression
with UnaryNode {
override def toString = s"${child}.$fieldName"
override def isAggregator = child.isAggregator
override lazy val resolved = false
override def dataType = throw new UnresolvedException(this, "field")
......@@ -445,7 +451,7 @@ object Expressions {
override def namedExpressions = child.namedExpressions + (alias -> child)
override def toString = s"$child as $alias"
override def isAggregator = child.isAggregator
lazy val dataType = {
if (!resolved) {
throw new UnresolvedException(this,
......@@ -519,6 +525,14 @@ object Expressions {
def listLiteral[_ <: PrimitiveType[_]](typ: ArrayType, rawValue: List[Expressions.Literal[_]]) = new ListLiteral(typ, rawValue)
def count() = new CountExpression()
def maxExpr(maxClause: Expression) = new MaxExpression(maxClause)
def minExpr(minClause: Expression) = new MinExpression(minClause)
def sumExpr(sumClause: Expression) = new SumExpression(sumClause)
case class ArithmeticExpression(symbol: String,
left: Expression,
right: Expression)
......@@ -681,13 +695,25 @@ object Expressions {
override def toString = s"$child where $condExpr"
}
case class SelectExpression(child: Expression, selectList: List[Expression]) extends Expression {
case class SelectExpression(child: Expression, selectList: List[Expression], forGroupBy: Boolean = false) extends Expression {
val children = List(child) ::: selectList
def hasAggregation = {
var result = false;
selectList.foreach { expr =>
{
result = result || expr.isAggregator
}
}
result
}
lazy val selectListWithAlias = selectList.zipWithIndex map {
case (s: AliasExpression, _) => s
case (x, i) => new AliasExpression(x, s"${x}")
}
lazy val dataType = {
if (!resolved) {
throw new UnresolvedException(this,
......@@ -698,7 +724,14 @@ object Expressions {
override def namedExpressions = child.namedExpressions ++ (selectList.flatMap(_.namedExpressions))
override def toString = s"""$child select ${selectListWithAlias.mkString("", ", ", "")}"""
override def toString = {
//When this is part of a group by, the child is only present so that the select
//list gets translated correctly. It is not really part of the query. The child
//ends up both in the GroupByExpression as well as here. We only want to show it
//in the GroupByExpression. Hide it here.
var prefix = if(forGroupBy) { "" } else { s"""${child} select """ }
s"""${prefix}${selectListWithAlias.mkString("", ", ", "")}"""
}
}
case class LoopExpression(val input: Expression, val loopingExpression: Expression,
......@@ -797,4 +830,68 @@ object Expressions {
child.dataType
}
}
case class CountExpression() extends Expression {
override def isAggregator = true
override def toString = s"count()"
val children = Nil
lazy val dataType = {
DataTypes.LONG_TYPE
}
}
case class MaxExpression(maxClause: Expression) extends Expression {
override def toString = s"max($maxClause)"
override def isAggregator = true
val children = List(maxClause)
lazy val dataType = {
if (!resolved) {
throw new UnresolvedException(this,
s"datatype. Can not resolve due to unresolved children")
}
maxClause.dataType
}
}
case class MinExpression(minClause: Expression) extends Expression {
override def toString = s"min($minClause)"
override def isAggregator = true
val children = List(minClause)
lazy val dataType = {
if (!resolved) {
throw new UnresolvedException(this,
s"datatype. Can not resolve due to unresolved children")
}
minClause.dataType
}
}
case class SumExpression(sumClause: Expression) extends Expression {
override def toString = s"sum($sumClause)"
override def isAggregator = true
val children = List(sumClause)
lazy val dataType = {
if (!resolved) {
throw new UnresolvedException(this,
s"datatype. Can not resolve due to unresolved children")
}
sumClause.dataType
}
}
case class GroupByExpression(child: Expression, groupBy: SelectExpression, selExpr: SelectExpression) extends Expression{
override def toString = s"from ${child} groupby(${groupBy}) select ${selExpr}"
val children = List(child, groupBy, selExpr)
lazy val dataType = {
if (!resolved) {
throw new UnresolvedException(this,
s"datatype. Can not resolve due to unresolved children")
}
selExpr.dataType
}
}
}
......@@ -90,7 +90,7 @@ class GremlinEvaluator(qry: GremlinQuery, persistenceStrategy: GraphPersistenceS
if(debug) {
println(" rawRes " +rawRes)
}
if (!qry.hasSelectList) {
if (!qry.hasSelectList && ! qry.isGroupBy) {
val rows = rawRes.asInstanceOf[java.util.List[AnyRef]].map { v =>
val instObj = instanceObject(v)
val o = persistenceStrategy.constructInstance(oType, instObj)
......@@ -112,6 +112,19 @@ class GremlinEvaluator(qry: GremlinQuery, persistenceStrategy: GraphPersistenceS
sInstance.set(cName, persistenceStrategy.constructInstance(aE.dataType, v))
}
}
else if(qry.isGroupBy) {
//the order in the result will always match the order in the select list
val selExpr = qry.expr.asInstanceOf[GroupByExpression].selExpr
var idx = 0;
val row : java.util.List[Object] = rV.asInstanceOf[java.util.List[Object]]
selExpr.selectListWithAlias.foreach { aE =>
val cName = aE.alias
val cValue = row.get(idx);
sInstance.set(cName, persistenceStrategy.constructInstance(aE.dataType, cValue))
idx += 1;
}
}
addPathStruct(r, sInstance)
}
GremlinQueryResult(qry.expr.toString, rType, rows.toList)
......
......@@ -24,6 +24,8 @@ import scala.util.parsing.combinator.lexical.StdLexical
import scala.util.parsing.combinator.syntactical.StandardTokenParsers
import scala.util.parsing.combinator.{ImplicitConversions, PackratParsers}
import scala.util.parsing.input.CharArrayReader._
import org.apache.atlas.AtlasException
import org.apache.atlas.typesystem.types.DataTypes
trait QueryKeywords {
this: StandardTokenParsers =>
......@@ -67,6 +69,10 @@ trait QueryKeywords {
protected val LIMIT = Keyword("limit")
protected val OFFSET = Keyword("offset")
protected val ORDERBY = Keyword("orderby")
protected val COUNT = Keyword("count")
protected val MAX = Keyword("max")
protected val MIN = Keyword("min")
protected val SUM = Keyword("sum")
}
trait ExpressionUtils {
......@@ -79,14 +85,14 @@ trait ExpressionUtils {
case (c, t, Some(a)) => input.loop(c, t.get).as(a)
}
def select(input: Expression, s: List[(Expression, Option[String])]) = {
def select(input: Expression, s: List[(Expression, Option[String])], forGroupBy: Boolean = false) = {
val selList = s.map { t =>
t._2 match {
case None => t._1.as(s"${t._1}")
case _ => t._1.as(t._2.get)
}
}
input.select(selList: _*)
new SelectExpression(input, selList, forGroupBy)
}
def limit(input: Expression, lmt: Literal[Integer], offset: Literal[Integer]) = {
......@@ -117,6 +123,10 @@ trait ExpressionUtils {
val leftSrcId = leftmostId(sngQuery2)
sngQuery2.transformUp(replaceIdWithField(leftSrcId, snglQuery1.field(leftSrcId.name)))
}
def groupBy(input: Expression, groupByExpr: SelectExpression, selectExpr: SelectExpression) = {
input.groupBy(groupByExpr, selectExpr)
}
}
case class QueryParams(limit: Int, offset: Int)
......@@ -164,8 +174,8 @@ object QueryParser extends StandardTokenParsers with QueryKeywords with Expressi
*
* @return
*/
def query(implicit queryParams: QueryParams) = querySrc ~ opt(loopExpression) ~ opt(selectClause) ~ opt(orderby) ~ opt(limitOffset) ^^ {
case s ~ l ~ sel ~ odr ~ lmtoff => {
def query(implicit queryParams: QueryParams) = querySrc ~ opt(loopExpression) ~ opt(groupByExpr) ~ opt(selectClause) ~ opt(orderby) ~ opt(limitOffset) ^^ {
case s ~ l ~ grp ~ sel ~ odr ~ lmtoff => {
var expressiontree = s
if (l.isDefined) //Note: The order of if statements is important.
{
......@@ -175,10 +185,6 @@ object QueryParser extends StandardTokenParsers with QueryKeywords with Expressi
{
expressiontree = order(expressiontree, odr.get._1, odr.get._2)
}
if (sel.isDefined)
{
expressiontree = select(expressiontree, sel.get)
}
if (queryParams != null && lmtoff.isDefined)
{
val mylimit = int(min(queryParams.limit, max(lmtoff.get._1 - queryParams.offset, 0)))
......@@ -189,6 +195,34 @@ object QueryParser extends StandardTokenParsers with QueryKeywords with Expressi
} else if(queryParams != null) {
expressiontree = limit(expressiontree, int(queryParams.limit), int(queryParams.offset))
}
if (grp.isDefined && sel.isDefined)
{
var child = expressiontree
var selectExpr: SelectExpression = select(child, sel.get, true)
var grpBySelectExpr: SelectExpression = select(child, grp.get, true)
expressiontree = groupBy(child, grpBySelectExpr, selectExpr)
}
else if (grp.isDefined)
{
throw new AtlasException("groupby without select is not allowed");
}
else if (sel.isDefined)
{
var selectChild = expressiontree
val selExpr : SelectExpression = select(selectChild, sel.get);
if(selExpr.hasAggregation) {
//In order to do the aggregation, we need to add an implicit group by. Having the
//group by expression be a constant values forces all of the vertices into one group.
val groupByConstant : Expression = Expressions.literal(DataTypes.STRING_TYPE, "dummy");
val groupBySelExpr : SelectExpression = select(selectChild, sel.get, true);
val groupByListExpr : SelectExpression = select(selectChild, List((groupByConstant,None)), true)
expressiontree = groupBy(selectChild, groupByListExpr, groupBySelExpr)
}
else {
expressiontree = selExpr
}
}
expressiontree
}
}
......@@ -279,7 +313,7 @@ object QueryParser extends StandardTokenParsers with QueryKeywords with Expressi
arithE ~ (LT | LTE | EQ | NEQ | GT | GTE) ~ arithE ^^ { case l ~ op ~ r => l.compareOp(op)(r)} |
arithE ~ (ISA | IS) ~ ident ^^ { case l ~ i ~ t => l.isTrait(t)} |
arithE ~ HAS ~ ident ^^ { case l ~ i ~ f => l.hasField(f)} |
arithE
arithE | countClause | maxClause | minClause | sumClause
def arithE = multiE ~ opt(rep(arithERight)) ^^ {
case l ~ None => l
......@@ -351,6 +385,22 @@ object QueryParser extends StandardTokenParsers with QueryKeywords with Expressi
def doubleConstant: Parser[String] =
elem("int", _.isInstanceOf[lexical.DoubleLiteral]) ^^ (_.chars)
def countClause = COUNT ~ LPAREN ~ RPAREN ^^ {
case c => count()
}
def maxClause = MAX ~ (LPAREN ~> expr <~ RPAREN) ^^ {
case m ~ e => maxExpr(e)
}
def minClause = MIN ~ (LPAREN ~> expr <~ RPAREN) ^^ {
case m ~ e => minExpr(e)
}
def sumClause = SUM ~ (LPAREN ~> expr <~ RPAREN) ^^ {
case m ~ e => sumExpr(e)
}
def groupByExpr = GROUPBY ~ (LPAREN ~> rep1sep(selectExpression, COMMA) <~ RPAREN) ^^ {
case g ~ ce => ce
}
}
class QueryLexer(val keywords: Seq[String], val delims: Seq[String]) extends StdLexical with ImplicitConversions {
......
......@@ -81,11 +81,11 @@ class Resolver(srcExpr: Option[Expression] = None, aliases: Map[String, Expressi
val r = new Resolver(Some(inputExpr), inputExpr.namedExpressions)
return new FilterExpression(inputExpr, condExpr.transformUp(r))
}
case SelectExpression(child, selectList) if child.resolved => {
case SelectExpression(child, selectList, forGroupBy) if child.resolved => {
val r = new Resolver(Some(child), child.namedExpressions)
return new SelectExpression(child, selectList.map {
_.transformUp(r)
})
}, forGroupBy)
}
case l@LoopExpression(inputExpr, loopExpr, t) if inputExpr.resolved => {
val r = new Resolver(Some(inputExpr), inputExpr.namedExpressions, true)
......@@ -145,11 +145,15 @@ object FieldValidator extends PartialFunction[Expression, Expression] {
new FilterExpression(inputExpr, validatedCE)
}
}
case SelectExpression(child, selectList) if child.resolved => {
case SelectExpression(child, selectList, forGroupBy) if child.resolved => {
val v = validateQualifiedField(child.dataType)
return new SelectExpression(child, selectList.map {
_.transformUp(v)
})
}, forGroupBy)
}
case OrderExpression(child, order, asc) => {
val v = validateQualifiedField(child.dataType)
OrderExpression(child, order.transformUp(v), asc)
}
case l@LoopExpression(inputExpr, loopExpr, t) => {
val validatedLE = loopExpr.transformUp(validateQualifiedField(inputExpr.dataType))
......
......@@ -66,15 +66,16 @@ public class BaseRepositoryTest {
protected MetadataRepository repository;
protected void setUp() throws Exception {
//force graph initialization / built in type registration
TestUtils.getGraph();
setUpDefaultTypes();
setUpTypes();
TestUtils.getGraph().commit();
new GraphBackedSearchIndexer(new AtlasTypeRegistry());
TestUtils.resetRequestContext();
setupInstances();
TestUtils.getGraph().commit();
TestUtils.dumpGraph(TestUtils.getGraph());
}
......
......@@ -156,7 +156,8 @@ public final class TestUtils {
createOptionalAttrDef("salary", DataTypes.DOUBLE_TYPE),
createOptionalAttrDef("age", DataTypes.FLOAT_TYPE),
createOptionalAttrDef("numberOfStarsEstimate", DataTypes.BIGINTEGER_TYPE),
createOptionalAttrDef("approximationOfPi", DataTypes.BIGDECIMAL_TYPE)
createOptionalAttrDef("approximationOfPi", DataTypes.BIGDECIMAL_TYPE),
createOptionalAttrDef("isOrganDonor", DataTypes.BOOLEAN_TYPE)
);
HierarchicalTypeDefinition<ClassType> managerTypeDef = createClassTypeDef("Manager", "Manager"+_description, ImmutableSet.of("Person"),
......@@ -195,6 +196,7 @@ public final class TestUtils {
john.set("address", johnAddr);
john.set("birthday",new Date(1950, 5, 15));
john.set("isOrganDonor", true);
john.set("hasPets", true);
john.set("numberOfCars", 1);
john.set("houseNumber", 153);
......@@ -227,6 +229,7 @@ public final class TestUtils {
max.set("manager", jane);
max.set("mentor", julius);
max.set("birthday",new Date(1979, 3, 15));
max.set("isOrganDonor", true);
max.set("hasPets", true);
max.set("age", 36);
max.set("numberOfCars", 2);
......
Markdown is supported
0% or
You are about to add 0 people to the discussion. Proceed with caution.
Finish editing this message first!
Please register or to comment