{ "cells": [ { "cell_type": "markdown", "metadata": {}, "source": [ "Nessie Iceberg/Flink SQL Demo with NBA Dataset\n", "============================\n", "This demo showcases how to use Nessie Python API along with Flink from Iceberg\n", "\n", "Initialize PyFlink\n", "----------------------------------------------\n", "To get started, we will first have to do a few setup steps that give us everything we need\n", "to get started with Nessie. In case you're interested in the detailed setup steps for Flink, you can check out the [docs](https://projectnessie.org/tools/iceberg/flink/)\n", "\n", "The Binder server has downloaded flink and some data for us as well as started a Nessie server in the background. All we have to do is start Flink\n", "\n", "The below cell starts a local Flink session with parameters needed to configure Nessie. Each config option is followed by a comment explaining its purpose." ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "import os\n", "from pyflink.datastream import StreamExecutionEnvironment\n", "from pyflink.table import StreamTableEnvironment\n", "from pyflink.table.expressions import lit\n", "from pynessie import init\n", "\n", "# where we will store our data\n", "warehouse = os.path.join(os.getcwd(), \"flink-warehouse\")\n", "# this was downloaded when Binder started, its available on maven central\n", "iceberg_flink_runtime_jar = os.path.join(os.getcwd(), \"../iceberg-flink-runtime-1.17-1.4.2.jar\")\n", "assert os.path.exists(iceberg_flink_runtime_jar)\n", "\n", "env = StreamExecutionEnvironment.get_execution_environment()\n", "env.add_jars(\"file://{}\".format(iceberg_flink_runtime_jar))\n", "table_env = StreamTableEnvironment.create(env)\n", "\n", "nessie_client = init()\n", "\n", "\n", "def create_ref_catalog(ref):\n", " \"\"\"\n", " Create a flink catalog that is tied to a specific ref.\n", "\n", " In order to create the catalog we have to first create the branch\n", " \"\"\"\n", " default_branch = nessie_client.get_default_branch()\n", " if ref != default_branch:\n", " default_branch_hash = nessie_client.get_reference(default_branch).hash_\n", " nessie_client.create_branch(ref, ref=default_branch, hash_on_ref=default_branch_hash)\n", " # The important args below are:\n", " # type - tell Flink to use Iceberg as the catalog\n", " # catalog-impl - which Iceberg catalog to use, in this case we want Nessie\n", " # uri - the location of the nessie server.\n", " # ref - the Nessie ref/branch we want to use (defaults to main)\n", " # warehouse - the location this catalog should store its data\n", " table_env.execute_sql(\n", " f\"\"\"CREATE CATALOG {ref}_catalog WITH (\n", " 'type'='iceberg',\n", " 'catalog-impl'='org.apache.iceberg.nessie.NessieCatalog',\n", " 'uri'='http://localhost:19120/api/v1',\n", " 'ref'='{ref}',\n", " 'warehouse' = '{warehouse}')\"\"\"\n", " )\n", "\n", "\n", "create_ref_catalog(nessie_client.get_default_branch())\n", "print(\"\\n\\n\\nFlink running\\n\\n\\n\")\n", "\n", "# Create the 'nba' namespace.\n", "table_env.execute_sql(\"CREATE DATABASE main_catalog.nba\").wait()" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "Solving Data Engineering problems with Nessie\n", "============================\n", "\n", "In this Demo we are a data engineer working at a fictional sports analytics blog. In order for the authors to write articles they have to have access to the relevant data. They need to be able to retrieve data quickly and be able to create charts with it.\n", "\n", "We have been asked to collect and expose some information about basketball players. We have located some data sources and are now ready to start ingesting data into our data lakehouse. We will perform the ingestion steps on a Nessie branch to test and validate the data before exposing to the analysts." ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "Set up Nessie branches (via Nessie CLI)\n", "----------------------------\n", "Once all dependencies are configured, we can get started with ingesting our basketball data into `Nessie` with the following steps:\n", "\n", "- Create a new branch named `dev`\n", "- List all branches\n", "\n", "It is worth mentioning that we don't have to explicitly create a `main` branch, since it's the default branch." ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "create_ref_catalog(\"dev\")" ] }, { "cell_type": "markdown", "metadata": { "pycharm": { "name": "#%% md\n" } }, "source": [ "We have created the branch `dev` and we can see the branch with the Nessie `hash` its currently pointing to.\n", "\n", "Below we list all branches. Note that the auto created `main` branch already exists and both branches point at the same empty `hash` initially" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "!nessie --verbose branch" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "Create tables under dev branch\n", "-------------------------------------\n", "Once we created the `dev` branch and verified that it exists, we can create some tables and add some data.\n", "\n", "We create two tables under the `dev` branch:\n", "- `salaries`\n", "- `totals_stats`\n", "\n", "These tables list the salaries per player per year and their stats per year.\n", "\n", "To create the data we:\n", "\n", "1. switch our branch context to dev\n", "2. create the table\n", "3. insert the data from an existing csv file. This csv file is already stored locally on the demo machine. A production use case would likely take feeds from official data sources" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "# Load the dataset\n", "from pyflink.table import DataTypes, Schema, TableDescriptor\n", "from pyflink.table.expressions import col\n", "\n", "# Creating `salaries` table\n", "(\n", " table_env.create_temporary_table(\n", " \"dev_catalog.nba.salaries_temp\",\n", " TableDescriptor.for_connector(\"filesystem\")\n", " .schema(\n", " Schema.new_builder()\n", " .column(\"Season\", DataTypes.STRING())\n", " .column(\"Team\", DataTypes.STRING())\n", " .column(\"Salary\", DataTypes.STRING())\n", " .column(\"Player\", DataTypes.STRING())\n", " .build()\n", " )\n", " .option(\"path\", \"../datasets/nba/salaries.csv\")\n", " .format(\"csv\")\n", " .build(),\n", " )\n", ")\n", "\n", "table_env.execute_sql(\n", " \"\"\"CREATE TABLE IF NOT EXISTS dev_catalog.nba.`salaries@dev`\n", " (Season STRING, Team STRING, Salary STRING, Player STRING)\"\"\"\n", ").wait()\n", "\n", "tab = table_env.from_path(\"dev_catalog.nba.salaries_temp\")\n", "tab.execute_insert(\"dev_catalog.nba.`salaries@dev`\").wait()\n", "\n", "# Creating `totals_stats` table\n", "(\n", " table_env.create_temporary_table(\n", " \"dev_catalog.nba.totals_stats_temp\",\n", " TableDescriptor.for_connector(\"filesystem\")\n", " .schema(\n", " Schema.new_builder()\n", " .column(\"Season\", DataTypes.STRING())\n", " .column(\"Age\", DataTypes.STRING())\n", " .column(\"Team\", DataTypes.STRING())\n", " .column(\"ORB\", DataTypes.STRING())\n", " .column(\"DRB\", DataTypes.STRING())\n", " .column(\"TRB\", DataTypes.STRING())\n", " .column(\"AST\", DataTypes.STRING())\n", " .column(\"STL\", DataTypes.STRING())\n", " .column(\"BLK\", DataTypes.STRING())\n", " .column(\"TOV\", DataTypes.STRING())\n", " .column(\"PTS\", DataTypes.STRING())\n", " .column(\"Player\", DataTypes.STRING())\n", " .column(\"RSorPO\", DataTypes.STRING())\n", " .build()\n", " )\n", " .option(\"path\", \"../datasets/nba/totals_stats.csv\")\n", " .format(\"csv\")\n", " .build(),\n", " )\n", ")\n", "\n", "table_env.execute_sql(\n", " \"\"\"CREATE TABLE IF NOT EXISTS dev_catalog.nba.`totals_stats@dev` (Season STRING, Age STRING, Team STRING,\n", " ORB STRING, DRB STRING, TRB STRING, AST STRING, STL STRING, BLK STRING, TOV STRING, PTS STRING,\n", " Player STRING, RSorPO STRING)\"\"\"\n", ").wait()\n", "\n", "tab = table_env.from_path(\"dev_catalog.nba.totals_stats_temp\")\n", "tab.execute_insert(\"dev_catalog.nba.`totals_stats@dev`\").wait()\n", "\n", "salaries = table_env.from_path(\"dev_catalog.nba.`salaries@dev`\").select(col(\"Season\").count).to_pandas().values[0][0]\n", "totals_stats = (\n", " table_env.from_path(\"dev_catalog.nba.`totals_stats@dev`\").select(col(\"Season\").count).to_pandas().values[0][0]\n", ")\n", "print(f\"\\n\\n\\nAdded {salaries} rows to the salaries table and {totals_stats} rows to the totals_stats table.\\n\\n\\n\")" ] }, { "cell_type": "markdown", "metadata": { "pycharm": { "name": "#%% md\n" } }, "source": [ "Now we count the rows in our tables to ensure they are the same number as the csv files. Note we use the `table@branch` notation which overrides the context set by the catalog." ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "table_count = table_env.from_path(\"dev_catalog.nba.`salaries@dev`\").select(col(\"Season\").count).to_pandas().values[0][0]\n", "csv_count = table_env.from_path(\"dev_catalog.nba.salaries_temp\").select(col(\"Season\").count).to_pandas().values[0][0]\n", "assert table_count == csv_count\n", "print(table_count)\n", "\n", "table_count = (\n", " table_env.from_path(\"dev_catalog.nba.`totals_stats@dev`\").select(col(\"Season\").count).to_pandas().values[0][0]\n", ")\n", "csv_count = (\n", " table_env.from_path(\"dev_catalog.nba.totals_stats_temp\").select(col(\"Season\").count).to_pandas().values[0][0]\n", ")\n", "assert table_count == csv_count\n", "print(table_count)" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "Check generated tables\n", "----------------------------\n", "Since we have been working solely on the `dev` branch, where we created 2 tables and added some data,\n", "let's verify that the `main` branch was not altered by our changes." ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "!nessie content list" ] }, { "cell_type": "markdown", "metadata": { "pycharm": { "name": "#%% md\n" } }, "source": [ "And on the `dev` branch we expect to see two tables" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "!nessie content list --ref dev" ] }, { "cell_type": "markdown", "metadata": { "pycharm": { "name": "#%% md\n" } }, "source": [ "We can also verify that the `dev` and `main` branches point to different commits" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "!nessie --verbose branch" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "Dev promotion into main\n", "-----------------------\n", "Once we are done with our changes on the `dev` branch, we would like to merge those changes into `main`.\n", "We merge `dev` into `main` via the command line `merge` command.\n", "Both branches should be at the same revision after merging/promotion." ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "!nessie merge dev -b main --force" ] }, { "cell_type": "markdown", "metadata": { "pycharm": { "name": "#%% md\n" } }, "source": [ "We can verify that the `main` branch now contains the expected tables and row counts.\n", "\n", "The tables are now on `main` and ready for consumption by our blog authors and analysts!" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "!nessie --verbose branch" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "!nessie content list" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "table_count = table_env.from_path(\"main_catalog.nba.salaries\").select(col(\"Season\").count).to_pandas().values[0][0]\n", "csv_count = table_env.from_path(\"dev_catalog.nba.salaries_temp\").select(col(\"Season\").count).to_pandas().values[0][0]\n", "assert table_count == csv_count\n", "\n", "table_count = table_env.from_path(\"main_catalog.nba.totals_stats\").select(col(\"Season\").count).to_pandas().values[0][0]\n", "csv_count = (\n", " table_env.from_path(\"dev_catalog.nba.totals_stats_temp\").select(col(\"Season\").count).to_pandas().values[0][0]\n", ")\n", "assert table_count == csv_count" ] }, { "cell_type": "markdown", "metadata": { "pycharm": { "name": "#%% md\n" } }, "source": [ "Perform regular ETL on the new tables\n", "-------------------\n", "Our analysts are happy with the data and we want to now regularly ingest data to keep things up to date. Our first ETL job consists of the following:\n", "\n", "1. Update the salaries table to add new data\n", "2. We have decided the `Age` column isn't required in the `totals_stats` table so we will drop the column\n", "3. We create a new table to hold information about the players appearances in all star games\n", "\n", "As always we will do this work on a branch and verify the results. This ETL job can then be set up to run nightly with new stats and salary information." ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "create_ref_catalog(\"etl\")" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "# add some salaries for Kevin Durant\n", "table_env.execute_sql(\n", " \"\"\"INSERT INTO etl_catalog.nba.salaries\n", " VALUES ('2017-18', 'Golden State Warriors', '$25000000', 'Kevin Durant'),\n", " ('2018-19', 'Golden State Warriors', '$30000000', 'Kevin Durant'),\n", " ('2019-20', 'Brooklyn Nets', '$37199000', 'Kevin Durant'),\n", " ('2020-21', 'Brooklyn Nets', '$39058950', 'Kevin Durant')\"\"\"\n", ").wait()" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "# Rename the table `totals_stats` to `new_totals_stats`\n", "table_env.execute_sql(\n", " \"ALTER TABLE etl_catalog.nba.`totals_stats@etl` RENAME TO etl_catalog.nba.new_totals_stats\"\n", ").wait()" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "# Creating `allstar_games_stats` table\n", "(\n", " table_env.create_temporary_table(\n", " \"etl_catalog.nba.allstar_games_stats_temp\",\n", " TableDescriptor.for_connector(\"filesystem\")\n", " .schema(\n", " Schema.new_builder()\n", " .column(\"Season\", DataTypes.STRING())\n", " .column(\"Age\", DataTypes.STRING())\n", " .column(\"Team\", DataTypes.STRING())\n", " .column(\"ORB\", DataTypes.STRING())\n", " .column(\"TRB\", DataTypes.STRING())\n", " .column(\"AST\", DataTypes.STRING())\n", " .column(\"STL\", DataTypes.STRING())\n", " .column(\"BLK\", DataTypes.STRING())\n", " .column(\"TOV\", DataTypes.STRING())\n", " .column(\"PF\", DataTypes.STRING())\n", " .column(\"PTS\", DataTypes.STRING())\n", " .column(\"Player\", DataTypes.STRING())\n", " .build()\n", " )\n", " .option(\"path\", \"../datasets/nba/allstar_games_stats.csv\")\n", " .format(\"csv\")\n", " .build(),\n", " )\n", ")\n", "\n", "table_env.execute_sql(\n", " \"\"\"CREATE TABLE IF NOT EXISTS etl_catalog.nba.`allstar_games_stats@etl` (Season STRING, Age STRING,\n", " Team STRING, ORB STRING, TRB STRING, AST STRING, STL STRING, BLK STRING, TOV STRING,\n", " PF STRING, PTS STRING, Player STRING)\"\"\"\n", ").wait()\n", "\n", "tab = table_env.from_path(\"etl_catalog.nba.allstar_games_stats_temp\")\n", "tab.execute_insert(\"etl_catalog.nba.`allstar_games_stats@etl`\").wait()\n", "\n", "# Notice how we view the data on the etl branch via @etl\n", "table_env.from_path(\"etl_catalog.nba.`allstar_games_stats@etl`\").to_pandas()" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "We can verify that the new table isn't on the `main` branch but is present on the etl branch" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "# Since we have been working on the `etl` branch, the `allstar_games_stats` table is not on the `main` branch\n", "!nessie content list" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "# We should see `allstar_games_stats` and the `new_totals_stats` on the `etl` branch\n", "!nessie content list --ref etl" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "Now that we are happy with the data we can again merge it into `main`" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "!nessie merge etl -b main --force" ] }, { "cell_type": "markdown", "metadata": { "pycharm": { "name": "#%% md\n" } }, "source": [ "Now lets verify that the changes exist on the `main` branch" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "!nessie content list" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "!nessie --verbose branch" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "table_count = (\n", " table_env.from_path(\"main_catalog.nba.allstar_games_stats\").select(col(\"Season\").count).to_pandas().values[0][0]\n", ")\n", "csv_count = (\n", " table_env.from_path(\"etl_catalog.nba.allstar_games_stats_temp\").select(col(\"Season\").count).to_pandas().values[0][0]\n", ")\n", "assert table_count == csv_count" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "Create `experiment` branch\n", "--------------------------------\n", "As a data analyst we might want to carry out some experiments with some data, without affecting `main` in any way.\n", "As in the previous examples, we can just get started by creating an `experiment` branch off of `main`\n", "and carry out our experiment, which could consist of the following steps:\n", "- drop `totals_stats` table\n", "- add data to `salaries` table\n", "- compare `experiment` and `main` tables" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "create_ref_catalog(\"experiment\")" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "# Drop the `new_totals_stats` table on the `experiment` branch\n", "table_env.execute_sql(\"DROP TABLE experiment_catalog.nba.`new_totals_stats@etl`\")" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "# add some salaries for Dirk Nowitzki\n", "table_env.execute_sql(\n", " \"\"\"INSERT INTO experiment_catalog.nba.salaries VALUES\n", " ('2015-16', 'Dallas Mavericks', '$8333333', 'Dirk Nowitzki'),\n", " ('2016-17', 'Dallas Mavericks', '$25000000', 'Dirk Nowitzki'),\n", " ('2017-18', 'Dallas Mavericks', '$5000000', 'Dirk Nowitzki'),\n", " ('2018-19', 'Dallas Mavericks', '$5000000', 'Dirk Nowitzki')\"\"\"\n", ").wait()" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "# We should see the `salaries` and `allstar_games_stats` tables only (since we just dropped `new_totals_stats`)\n", "!nessie content list --ref experiment" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "# `main` hasn't changed been changed and still has the `new_totals_stats` table\n", "!nessie content list" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "Let's take a look at the contents of the `salaries` table on the `experiment` branch.\n", "Notice the use of the `nessie` catalog and the use of `@experiment` to view data on the `experiment` branch" ] }, { "cell_type": "code", "execution_count": null, "metadata": {}, "outputs": [], "source": [ "table_env.from_path(\"main_catalog.nba.`salaries@experiment`\").select(lit(1).count).to_pandas()" ] }, { "cell_type": "markdown", "metadata": {}, "source": [ "and compare to the contents of the `salaries` table on the `main` branch. Notice that we didn't have to specify `@branchName` as it defaulted\n", "to the `main` branch" ] }, { "cell_type": "code", "execution_count": null, "metadata": { "pycharm": { "name": "#%%\n" } }, "outputs": [], "source": [ "table_env.from_path(\"main_catalog.nba.`salaries@main`\").select(lit(1).count).to_pandas()" ] } ], "metadata": { "kernelspec": { "display_name": "flink-demo", "language": "python", "name": "flink-demo" }, "language_info": { "codemirror_mode": { "name": "ipython", "version": 3 }, "file_extension": ".py", "mimetype": "text/x-python", "name": "python", "nbconvert_exporter": "python", "pygments_lexer": "ipython3", "version": "3.7.12" } }, "nbformat": 4, "nbformat_minor": 4 }