Skip to content

Commit

Permalink
Merge branch 'release/0.8.3'
Browse files Browse the repository at this point in the history
  • Loading branch information
metasim committed Oct 7, 2019
2 parents 14c8dc5 + eaaaf58 commit a8a8bb7
Show file tree
Hide file tree
Showing 68 changed files with 353 additions and 216 deletions.
33 changes: 20 additions & 13 deletions build/circleci/Dockerfile
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
FROM circleci/openjdk:8-jdk

ENV OPENJPEG_VERSION 2.3.0
ENV OPENJPEG_VERSION 2.3.1
ENV GDAL_VERSION 2.4.1
ENV JAVA_HOME /usr/lib/jvm/java-8-openjdk-amd64/

Expand All @@ -11,26 +11,37 @@ RUN sudo apt-get update && \
python python-minimal python2.7 python2.7-minimal \
libpython-stdlib libpython2.7 libpython2.7-minimal libpython2.7-stdlib \
&& sudo apt-get install -y \
python3 \
python3-pip \
pandoc \
wget \
gcc g++ build-essential \
libreadline-gplv2-dev libncursesw5-dev libssl-dev libsqlite3-dev tk-dev libgdbm-dev libc6-dev libbz2-dev \
libcurl4-gnutls-dev \
libproj-dev \
libgeos-dev \
libhdf4-alt-dev \
libhdf5-serial-dev \
bash-completion \
cmake \
imagemagick \
libpng-dev \
swig \
ant \
libffi-dev \
&& sudo apt autoremove \
&& sudo apt-get clean all \
&& pip3 install setuptools ipython==6.2.1 \
&& sudo update-alternatives --install /usr/bin/python python /usr/bin/python3 1
&& sudo apt-get clean all
# && sudo update-alternatives --install /usr/bin/python python /usr/bin/python3 1
# todo s

RUN cd /tmp && \
wget https://www.python.org/ftp/python/3.7.4/Python-3.7.4.tgz && \
tar xzf Python-3.7.4.tgz && \
cd Python-3.7.4 && \
./configure --with-ensurepip=install --prefix=/usr/local --enable-optimization && \
make && \
sudo make altinstall && \
rm -rf Python-3.7.4*

RUN sudo ln -s /usr/local/bin/python3.7 /usr/local/bin/python && \
sudo curl https://bootstrap.pypa.io/get-pip.py -o get-pip.py && \
sudo python get-pip.py && \
sudo pip3 install setuptools ipython==6.2.1

# install OpenJPEG
RUN cd /tmp && \
Expand All @@ -52,11 +63,9 @@ RUN cd /tmp && \
./configure \
--with-curl \
--with-hdf4 \
--with-hdf5 \
--with-geos \
--with-geotiff=internal \
--with-hide-internal-symbols \
--with-java=$JAVA_HOME \
--with-libtiff=internal \
--with-libz=internal \
--with-mrf \
Expand All @@ -68,7 +77,5 @@ RUN cd /tmp && \
&& \
make -j 8 && \
sudo make install && \
cd swig/java && \
sudo make install && \
sudo ldconfig && \
cd /tmp && sudo rm -Rf gdal*
2 changes: 2 additions & 0 deletions build/circleci/Makefile
Original file line number Diff line number Diff line change
@@ -0,0 +1,2 @@
all:
docker build -t "s22s/rasterframes-circleci:latest" .
2 changes: 1 addition & 1 deletion build/circleci/README.md
Original file line number Diff line number Diff line change
@@ -1,6 +1,6 @@
# CircleCI Dockerfile Build file

```bash
docker build -t s22s/rasterframes-circleci:latest .
make
docker push s22s/rasterframes-circleci:latest
```
3 changes: 3 additions & 0 deletions core/src/main/resources/reference.conf
Original file line number Diff line number Diff line change
Expand Up @@ -14,6 +14,9 @@ vlm.gdal {
AWS_REQUEST_PAYER = "requester"
GDAL_DISABLE_READDIR_ON_OPEN = "YES"
CPL_VSIL_CURL_ALLOWED_EXTENSIONS = ".tif,.tiff,.jp2,.mrf,.idx,.lrc,.mrf.aux.xml,.vrt"
GDAL_CACHEMAX = 512
GDAL_PAM_ENABLED = "NO"
CPL_VSIL_CURL_CHUNK_SIZE = 1000000
}
// set this to `false` if CPL_DEBUG is `ON`
useExceptions = true
Expand Down
8 changes: 6 additions & 2 deletions core/src/main/scala/org/apache/spark/sql/rf/TileUDT.scala
Original file line number Diff line number Diff line change
Expand Up @@ -26,6 +26,7 @@ import org.apache.spark.sql.types.{DataType, _}
import org.locationtech.rasterframes.encoders.CatalystSerializer
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.model.{Cells, TileDataContext}
import org.locationtech.rasterframes.ref.RasterRef.RasterRefTile
import org.locationtech.rasterframes.tiles.InternalRowTile


Expand Down Expand Up @@ -75,12 +76,15 @@ case object TileUDT {
implicit def tileSerializer: CatalystSerializer[Tile] = new CatalystSerializer[Tile] {

override val schema: StructType = StructType(Seq(
StructField("cell_context", schemaOf[TileDataContext], false),
StructField("cell_context", schemaOf[TileDataContext], true),
StructField("cell_data", schemaOf[Cells], false)
))

override def to[R](t: Tile, io: CatalystIO[R]): R = io.create(
io.to(TileDataContext(t)),
t match {
case _: RasterRefTile => null
case o => io.to(TileDataContext(o))
},
io.to(Cells(t))
)

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,18 +21,22 @@

package org.locationtech.rasterframes.expressions

import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors._
import com.typesafe.scalalogging.LazyLogging
import com.typesafe.scalalogging.Logger
import geotrellis.raster.Tile
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult.{TypeCheckFailure, TypeCheckSuccess}
import org.apache.spark.sql.catalyst.expressions.BinaryExpression
import org.apache.spark.sql.rf.TileUDT
import org.apache.spark.sql.types.DataType
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors._
import org.slf4j.LoggerFactory

/** Operation combining two tiles or a tile and a scalar into a new tile. */
trait BinaryLocalRasterOp extends BinaryExpression with LazyLogging {
trait BinaryLocalRasterOp extends BinaryExpression {

@transient protected lazy val logger = Logger(LoggerFactory.getLogger(getClass.getName))


override def dataType: DataType = left.dataType

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,18 +21,20 @@

package org.locationtech.rasterframes.expressions

import org.locationtech.rasterframes.expressions.DynamicExtractors.tileExtractor
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import com.typesafe.scalalogging.LazyLogging
import com.typesafe.scalalogging.Logger
import geotrellis.raster.Tile
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult.{TypeCheckFailure, TypeCheckSuccess}
import org.apache.spark.sql.catalyst.expressions.BinaryExpression
import org.apache.spark.sql.rf.TileUDT
import org.apache.spark.sql.types.DataType
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors.tileExtractor
import org.slf4j.LoggerFactory

/** Operation combining two tiles into a new tile. */
trait BinaryRasterOp extends BinaryExpression with LazyLogging {
trait BinaryRasterOp extends BinaryExpression {
@transient protected lazy val logger = Logger(LoggerFactory.getLogger(getClass.getName))

override def dataType: DataType = left.dataType

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,18 +21,20 @@

package org.locationtech.rasterframes.expressions

import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors._
import com.typesafe.scalalogging.LazyLogging
import com.typesafe.scalalogging.Logger
import geotrellis.raster.Tile
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult.{TypeCheckFailure, TypeCheckSuccess}
import org.apache.spark.sql.catalyst.expressions.UnaryExpression
import org.apache.spark.sql.rf.TileUDT
import org.apache.spark.sql.types.DataType
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors._
import org.slf4j.LoggerFactory

/** Operation on a tile returning a tile. */
trait UnaryLocalRasterOp extends UnaryExpression with LazyLogging {
trait UnaryLocalRasterOp extends UnaryExpression {
@transient protected lazy val logger = Logger(LoggerFactory.getLogger(getClass.getName))

override def dataType: DataType = child.dataType

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -37,10 +37,11 @@ trait UnaryRasterAggregate extends DeclarativeAggregate {
def children = Seq(child)

protected def tileOpAsExpression[R: TypeTag](name: String, op: Tile => R): Expression => ScalaUDF =
udfexpr[R, Any](name, (a: Any) => op(extractTileFromAny(a)))
udfexpr[R, Any](name, (a: Any) => if(a == null) null.asInstanceOf[R] else op(extractTileFromAny(a)))

protected val extractTileFromAny = (a: Any) => a match {
case t: Tile => t
case r: Row => rowTileExtractor(child.dataType)(r)._1
case null => null
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -21,33 +21,53 @@

package org.locationtech.rasterframes.expressions.accessors

import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.encoders.StandardEncoders.crsEncoder
import org.locationtech.rasterframes.expressions.OnTileContextExpression
import geotrellis.proj4.CRS
import org.apache.spark.sql.catalyst.InternalRow
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult.{TypeCheckFailure, TypeCheckSuccess}
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.catalyst.expressions.codegen.CodegenFallback
import org.apache.spark.sql.types.DataType
import org.apache.spark.sql.types.{DataType, StringType}
import org.apache.spark.sql.{Column, TypedColumn}
import org.locationtech.rasterframes.model.TileContext
import org.apache.spark.unsafe.types.UTF8String
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.encoders.StandardEncoders.crsEncoder
import org.locationtech.rasterframes.expressions.DynamicExtractors.projectedRasterLikeExtractor
import org.locationtech.rasterframes.model.LazyCRS

/**
* Expression to extract the CRS out of a RasterRef or ProjectedRasterTile column.
*
* @since 9/9/18
*/
@ExpressionDescription(
usage = "_FUNC_(raster) - Fetches the CRS of a ProjectedRasterTile or RasterSource.",
usage = "_FUNC_(raster) - Fetches the CRS of a ProjectedRasterTile or RasterSource, or converts a proj4 string column.",
examples = """
Examples:
> SELECT _FUNC_(raster);
....
""")
case class GetCRS(child: Expression) extends OnTileContextExpression with CodegenFallback {
case class GetCRS(child: Expression) extends UnaryExpression with CodegenFallback {
override def dataType: DataType = schemaOf[CRS]
override def nodeName: String = "rf_crs"
override def eval(ctx: TileContext): InternalRow = ctx.crs.toInternalRow

override def checkInputDataTypes(): TypeCheckResult = {
if (child.dataType != StringType && !projectedRasterLikeExtractor.isDefinedAt(child.dataType)) {
TypeCheckFailure(s"Input type '${child.dataType}' does not conform to `String` or `ProjectedRasterLike`.")
}
else TypeCheckSuccess
}

override protected def nullSafeEval(input: Any): Any = {
input match {
case s: UTF8String => LazyCRS(s.toString).toInternalRow
case row: InternalRow
val prl = projectedRasterLikeExtractor(child.dataType)(row)
prl.crs.toInternalRow
case o throw new IllegalArgumentException(s"Unsupported input type: $o")
}
}

}

object GetCRS {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,19 +21,20 @@

package org.locationtech.rasterframes.expressions.generators

import com.typesafe.scalalogging.LazyLogging
import com.typesafe.scalalogging.Logger
import org.apache.spark.sql.catalyst.InternalRow
import org.apache.spark.sql.catalyst.expressions._
import org.apache.spark.sql.catalyst.expressions.codegen.CodegenFallback
import org.apache.spark.sql.types.{DataType, StructField, StructType}
import org.apache.spark.sql.{Column, TypedColumn}
import org.locationtech.rasterframes
import org.locationtech.rasterframes.RasterSourceType
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.generators.RasterSourceToRasterRefs.bandNames
import org.locationtech.rasterframes.model.TileDimensions
import org.locationtech.rasterframes.tiles.ProjectedRasterTile
import org.locationtech.rasterframes.util._
import org.locationtech.rasterframes.RasterSourceType
import org.slf4j.LoggerFactory

import scala.util.Try
import scala.util.control.NonFatal
Expand All @@ -45,7 +46,9 @@ import scala.util.control.NonFatal
* @since 9/6/18
*/
case class RasterSourceToTiles(children: Seq[Expression], bandIndexes: Seq[Int], subtileDims: Option[TileDimensions] = None) extends Expression
with Generator with CodegenFallback with ExpectsInputTypes with LazyLogging {
with Generator with CodegenFallback with ExpectsInputTypes {

@transient protected lazy val logger = Logger(LoggerFactory.getLogger(getClass.getName))

override def inputTypes: Seq[DataType] = Seq.fill(children.size)(RasterSourceType)
override def nodeName: String = "rf_raster_source_to_tiles"
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,10 +21,7 @@

package org.locationtech.rasterframes.expressions.transformers

import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors._
import org.locationtech.rasterframes.expressions.row
import com.typesafe.scalalogging.LazyLogging
import com.typesafe.scalalogging.Logger
import geotrellis.raster
import geotrellis.raster.Tile
import geotrellis.raster.mapalgebra.local.{Defined, InverseMask => gtInverseMask, Mask => gtMask}
Expand All @@ -35,9 +32,16 @@ import org.apache.spark.sql.catalyst.expressions.{Expression, ExpressionDescript
import org.apache.spark.sql.rf.TileUDT
import org.apache.spark.sql.types.DataType
import org.apache.spark.sql.{Column, TypedColumn}
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors._
import org.locationtech.rasterframes.expressions.row
import org.slf4j.LoggerFactory

abstract class Mask(val left: Expression, val middle: Expression, val right: Expression, inverse: Boolean)
extends TernaryExpression with CodegenFallback with Serializable with LazyLogging {
extends TernaryExpression with CodegenFallback with Serializable {

@transient protected lazy val logger = Logger(LoggerFactory.getLogger(getClass.getName))


override def children: Seq[Expression] = Seq(left, middle, right)

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,24 +21,27 @@

package org.locationtech.rasterframes.expressions.transformers

import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.row
import com.typesafe.scalalogging.LazyLogging
import com.typesafe.scalalogging.Logger
import org.apache.spark.sql.catalyst.expressions.codegen.CodegenFallback
import org.apache.spark.sql.catalyst.expressions.{ExpectsInputTypes, Expression, UnaryExpression}
import org.apache.spark.sql.rf._
import org.apache.spark.sql.types.DataType
import org.apache.spark.sql.{Column, TypedColumn}
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.row
import org.locationtech.rasterframes.ref.RasterRef
import org.locationtech.rasterframes.tiles.ProjectedRasterTile
import org.slf4j.LoggerFactory

/**
* Realizes a RasterRef into a Tile.
*
* @since 11/2/18
*/
case class RasterRefToTile(child: Expression) extends UnaryExpression
with CodegenFallback with ExpectsInputTypes with LazyLogging {
with CodegenFallback with ExpectsInputTypes {

@transient protected lazy val logger = Logger(LoggerFactory.getLogger(getClass.getName))

override def nodeName: String = "raster_ref_to_tile"

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,7 +21,7 @@

package org.locationtech.rasterframes.expressions.transformers

import com.typesafe.scalalogging.LazyLogging
import com.typesafe.scalalogging.Logger
import org.apache.spark.sql.Column
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult
import org.apache.spark.sql.catalyst.analysis.TypeCheckResult.{TypeCheckFailure, TypeCheckSuccess}
Expand All @@ -33,6 +33,7 @@ import org.apache.spark.sql.types._
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors._
import org.locationtech.rasterframes.expressions.row
import org.slf4j.LoggerFactory

@ExpressionDescription(
usage = "_FUNC_(tile, value) - Set the NoData value for the given tile.",
Expand All @@ -45,7 +46,8 @@ import org.locationtech.rasterframes.expressions.row
> SELECT _FUNC_(tile, 1.5);
..."""
)
case class SetNoDataValue(left: Expression, right: Expression) extends BinaryExpression with CodegenFallback with LazyLogging {
case class SetNoDataValue(left: Expression, right: Expression) extends BinaryExpression with CodegenFallback {
@transient protected lazy val logger = Logger(LoggerFactory.getLogger(getClass.getName))

override val nodeName: String = "rf_with_no_data"
override def dataType: DataType = left.dataType
Expand Down
Loading

0 comments on commit a8a8bb7

Please sign in to comment.