Skip to content

Commit

Permalink
Merge pull request #301 from s22s/feature/rgb-composites
Browse files Browse the repository at this point in the history
Adds rf_render_png and rf_rgb_composite.
  • Loading branch information
metasim authored Aug 23, 2019
2 parents 0506fb1 + 2a4a8b2 commit 8df0c97
Show file tree
Hide file tree
Showing 31 changed files with 794 additions and 425 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -22,6 +22,7 @@
package org.locationtech.rasterframes
import geotrellis.proj4.CRS
import geotrellis.raster.mapalgebra.local.LocalTileBinaryOp
import geotrellis.raster.render.ColorRamp
import geotrellis.raster.{CellType, Tile}
import geotrellis.vector.Extent
import org.apache.spark.annotation.Experimental
Expand All @@ -34,6 +35,7 @@ import org.locationtech.rasterframes.expressions.aggregates._
import org.locationtech.rasterframes.expressions.generators._
import org.locationtech.rasterframes.expressions.localops._
import org.locationtech.rasterframes.expressions.tilestats._
import org.locationtech.rasterframes.expressions.transformers.RenderPNG.{RenderCompositePNG, RenderColorRampPNG}
import org.locationtech.rasterframes.expressions.transformers._
import org.locationtech.rasterframes.model.TileDimensions
import org.locationtech.rasterframes.stats._
Expand Down Expand Up @@ -81,6 +83,10 @@ trait RasterFunctions {
def rf_assemble_tile(columnIndex: Column, rowIndex: Column, cellData: Column, tileCols: Int, tileRows: Int, ct: CellType): TypedColumn[Any, Tile] =
rf_convert_cell_type(TileAssembler(columnIndex, rowIndex, cellData, lit(tileCols), lit(tileRows)), ct).as(cellData.columnName).as[Tile](singlebandTileEncoder)

/** Create a Tile from a column of cell data with location indexes and perform cell conversion. */
def rf_assemble_tile(columnIndex: Column, rowIndex: Column, cellData: Column, tileCols: Int, tileRows: Int): TypedColumn[Any, Tile] =
TileAssembler(columnIndex, rowIndex, cellData, lit(tileCols), lit(tileRows))

/** Create a Tile from a column of cell data with location indexes. */
def rf_assemble_tile(columnIndex: Column, rowIndex: Column, cellData: Column, tileCols: Column, tileRows: Column): TypedColumn[Any, Tile] =
TileAssembler(columnIndex, rowIndex, cellData, tileCols, tileRows)
Expand Down Expand Up @@ -317,12 +323,24 @@ trait RasterFunctions {
ReprojectGeometry(sourceGeom, srcCRSCol, dstCRSCol)

/** Render Tile as ASCII string, for debugging purposes. */
def rf_render_ascii(col: Column): TypedColumn[Any, String] =
DebugRender.RenderAscii(col)
def rf_render_ascii(tile: Column): TypedColumn[Any, String] =
DebugRender.RenderAscii(tile)

/** Render Tile cell values as numeric values, for debugging purposes. */
def rf_render_matrix(col: Column): TypedColumn[Any, String] =
DebugRender.RenderMatrix(col)
def rf_render_matrix(tile: Column): TypedColumn[Any, String] =
DebugRender.RenderMatrix(tile)

/** Converts tiles in a column into PNG encoded byte array, using given ColorRamp to assign values to colors. */
def rf_render_png(tile: Column, colors: ColorRamp): TypedColumn[Any, Array[Byte]] =
RenderColorRampPNG(tile, colors)

/** Converts columns of tiles representing RGB channels into a PNG encoded byte array. */
def rf_render_png(red: Column, green: Column, blue: Column): TypedColumn[Any, Array[Byte]] =
RenderCompositePNG(red, green, blue)

/** Converts columns of tiles representing RGB channels into a single RGB packaged tile. */
def rf_rgb_composite(red: Column, green: Column, blue: Column): Column =
RGBComposite(red, green, blue)

/** Cellwise less than value comparison between two tiles. */
def rf_local_less(left: Column, right: Column): Column = Less(left, right)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -128,6 +128,9 @@ package object expressions {

registry.registerExpression[DebugRender.RenderAscii]("rf_render_ascii")
registry.registerExpression[DebugRender.RenderMatrix]("rf_render_matrix")
registry.registerExpression[RenderPNG.RenderCompositePNG]("rf_render_png")
registry.registerExpression[RGBComposite]("rf_rgb_composite")

registry.registerExpression[transformers.ReprojectGeometry]("st_reproject")
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -21,28 +21,29 @@

package org.locationtech.rasterframes.expressions.transformers

import org.locationtech.rasterframes.expressions.UnaryRasterOp
import org.locationtech.rasterframes.util.TileAsMatrix
import geotrellis.raster.Tile
import geotrellis.raster.render.ascii.AsciiArtEncoder
import geotrellis.raster.{Tile, isNoData}
import org.apache.spark.sql.catalyst.expressions.codegen.CodegenFallback
import org.apache.spark.sql.catalyst.expressions.{Expression, ExpressionDescription}
import org.apache.spark.sql.types.{DataType, StringType}
import org.apache.spark.sql.{Column, TypedColumn}
import org.apache.spark.unsafe.types.UTF8String
import org.locationtech.rasterframes.expressions.UnaryRasterOp
import org.locationtech.rasterframes.model.TileContext
import spire.syntax.cfor.cfor

abstract class DebugRender(asciiArt: Boolean) extends UnaryRasterOp
with CodegenFallback with Serializable {
override def dataType: DataType = StringType
with CodegenFallback with Serializable {
import org.locationtech.rasterframes.expressions.transformers.DebugRender.TileAsMatrix
override def dataType: DataType = StringType

override protected def eval(tile: Tile, ctx: Option[TileContext]): Any = {
UTF8String.fromString(if (asciiArt)
s"\n${tile.renderAscii(AsciiArtEncoder.Palette.NARROW)}\n"
else
s"\n${tile.renderMatrix(6)}\n"
)
}
override protected def eval(tile: Tile, ctx: Option[TileContext]): Any = {
UTF8String.fromString(if (asciiArt)
s"\n${tile.renderAscii(AsciiArtEncoder.Palette.NARROW)}\n"
else
s"\n${tile.renderMatrix(6)}\n"
)
}
}

object DebugRender {
Expand Down Expand Up @@ -75,4 +76,29 @@ object DebugRender {
def apply(tile: Column): TypedColumn[Any, String] =
new Column(RenderMatrix(tile.expr)).as[String]
}

implicit class TileAsMatrix(val tile: Tile) extends AnyVal {
def renderMatrix(significantDigits: Int): String = {
val ND = s"%${significantDigits+5}s".format(Double.NaN)
val fmt = s"% ${significantDigits+5}.${significantDigits}g"
val buf = new StringBuilder("[")
cfor(0)(_ < tile.rows, _ + 1) { row =>
if(row > 0) buf.append(' ')
buf.append('[')
cfor(0)(_ < tile.cols, _ + 1) { col =>
val v = tile.getDouble(col, row)
if (isNoData(v)) buf.append(ND)
else buf.append(fmt.format(v))

if (col < tile.cols - 1)
buf.append(',')
}
buf.append(']')
if (row < tile.rows - 1)
buf.append(",\n")
}
buf.append("]")
buf.toString()
}
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,102 @@
/*
* This software is licensed under the Apache 2 license, quoted below.
*
* Copyright 2019 Astraea, Inc.
*
* Licensed 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.
*
* SPDX-License-Identifier: Apache-2.0
*
*/

package org.locationtech.rasterframes.expressions.transformers

import geotrellis.raster.ArrayMultibandTile
import org.apache.spark.sql.Column
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.codegen.CodegenFallback
import org.apache.spark.sql.catalyst.expressions.{Expression, ExpressionDescription, TernaryExpression}
import org.apache.spark.sql.rf.TileUDT
import org.apache.spark.sql.types.DataType
import org.locationtech.rasterframes._
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.expressions.DynamicExtractors.tileExtractor
import org.locationtech.rasterframes.expressions.row
import org.locationtech.rasterframes.tiles.ProjectedRasterTile

/**
* Expression to combine the given tile columns into an 32-bit RGB composite.
* Tiles in each row will first be and-ed with 0xFF, bit shifted, and or-ed into a single 32-bit word.
* @param red tile column to represent red channel
* @param green tile column to represent green channel
* @param blue tile column to represent blue channel
*/
@ExpressionDescription(
usage = "_FUNC_(red, green, blue) - Combines the given tile columns into an 32-bit RGB composite.",
arguments = """
Arguments:
* red - tile column representing the red channel
* green - tile column representing the green channel
* blue - tile column representing the blue channel"""
)
case class RGBComposite(red: Expression, green: Expression, blue: Expression) extends TernaryExpression
with CodegenFallback {

override def nodeName: String = "rf_rgb_composite"

override def dataType: DataType = if(
red.dataType.conformsTo[ProjectedRasterTile] ||
blue.dataType.conformsTo[ProjectedRasterTile] ||
green.dataType.conformsTo[ProjectedRasterTile]
) red.dataType
else TileType

override def children: Seq[Expression] = Seq(red, green, blue)

override def checkInputDataTypes(): TypeCheckResult = {
if (!tileExtractor.isDefinedAt(red.dataType)) {
TypeCheckFailure(s"Red channel input type '${red.dataType}' does not conform to a raster type.")
}
else if (!tileExtractor.isDefinedAt(green.dataType)) {
TypeCheckFailure(s"Green channel input type '${green.dataType}' does not conform to a raster type.")
}
else if (!tileExtractor.isDefinedAt(blue.dataType)) {
TypeCheckFailure(s"Blue channel input type '${blue.dataType}' does not conform to a raster type.")
}
else TypeCheckSuccess
}

override protected def nullSafeEval(input1: Any, input2: Any, input3: Any): Any = {
val (r, rc) = tileExtractor(red.dataType)(row(input1))
val (g, gc) = tileExtractor(green.dataType)(row(input2))
val (b, bc) = tileExtractor(blue.dataType)(row(input3))

// Pick the first available TileContext, if any, and reassociate with the result
val ctx = Seq(rc, gc, bc).flatten.headOption
val composite = ArrayMultibandTile(
r.rescale(0, 255), g.rescale(0, 255), b.rescale(0, 255)
).color()
ctx match {
case Some(c) => c.toProjectRasterTile(composite).toInternalRow
case None =>
implicit val tileSer = TileUDT.tileSerializer
composite.toInternalRow
}
}
}

object RGBComposite {
def apply(red: Column, green: Column, blue: Column): Column =
new Column(RGBComposite(red.expr, green.expr, blue.expr))
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,79 @@
/*
* This software is licensed under the Apache 2 license, quoted below.
*
* Copyright 2019 Astraea, Inc.
*
* Licensed 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.
*
* SPDX-License-Identifier: Apache-2.0
*
*/

package org.locationtech.rasterframes.expressions.transformers

import geotrellis.raster.Tile
import geotrellis.raster.render.ColorRamp
import org.apache.spark.sql.catalyst.expressions.codegen.CodegenFallback
import org.apache.spark.sql.catalyst.expressions.{Expression, ExpressionDescription}
import org.apache.spark.sql.types.{BinaryType, DataType}
import org.apache.spark.sql.{Column, TypedColumn}
import org.locationtech.rasterframes.expressions.UnaryRasterOp
import org.locationtech.rasterframes.model.TileContext

/**
* Converts a tile into a PNG encoded byte array.
* @param child tile column
* @param ramp color ramp to use for non-composite tiles.
*/
abstract class RenderPNG(child: Expression, ramp: Option[ColorRamp]) extends UnaryRasterOp with CodegenFallback with Serializable {
override def dataType: DataType = BinaryType
override protected def eval(tile: Tile, ctx: Option[TileContext]): Any = {
val png = ramp.map(tile.renderPng).getOrElse(tile.renderPng())
png.bytes
}
}

object RenderPNG {
import org.locationtech.rasterframes.encoders.SparkBasicEncoders._

@ExpressionDescription(
usage = "_FUNC_(tile) - Encode the given tile into a RGB composite PNG. Assumes the red, green, and " +
"blue channels are encoded as 8-bit channels within the 32-bit word.",
arguments = """
Arguments:
* tile - tile to render"""
)
case class RenderCompositePNG(child: Expression) extends RenderPNG(child, None) {
override def nodeName: String = "rf_render_png"
}

object RenderCompositePNG {
def apply(red: Column, green: Column, blue: Column): TypedColumn[Any, Array[Byte]] =
new Column(RenderCompositePNG(RGBComposite(red.expr, green.expr, blue.expr))).as[Array[Byte]]
}

@ExpressionDescription(
usage = "_FUNC_(tile) - Encode the given tile as a PNG using a color ramp with assignemnts from quantile computation",
arguments = """
Arguments:
* tile - tile to render"""
)
case class RenderColorRampPNG(child: Expression, colors: ColorRamp) extends RenderPNG(child, Some(colors)) {
override def nodeName: String = "rf_render_png"
}

object RenderColorRampPNG {
def apply(tile: Column, colors: ColorRamp): TypedColumn[Any, Array[Byte]] =
new Column(RenderColorRampPNG(tile.expr, colors)).as[Array[Byte]]
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -30,6 +30,7 @@ import org.apache.spark.sql.{DataFrame, Row, SparkSession}
import org.locationtech.rasterframes._
import org.locationtech.rasterframes.encoders.CatalystSerializer._
import org.locationtech.rasterframes.model.TileDimensions
import org.locationtech.rasterframes.tiles.ProjectedRasterTile

trait SinglebandGeoTiffMethods extends MethodExtensions[SinglebandGeoTiff] {
def toDF(dims: TileDimensions = NOMINAL_TILE_DIMS)(implicit spark: SparkSession): DataFrame = {
Expand All @@ -56,4 +57,6 @@ trait SinglebandGeoTiffMethods extends MethodExtensions[SinglebandGeoTiff] {

spark.createDataFrame(spark.sparkContext.makeRDD(rows, 1), schema)
}

def toProjectedRasterTile: ProjectedRasterTile = ProjectedRasterTile(self.projectedRaster)
}
Loading

0 comments on commit 8df0c97

Please sign in to comment.