| |
| |
| <!DOCTYPE html> |
| <!--[if IE 8]><html class="no-js lt-ie9" lang="en" > <![endif]--> |
| <!--[if gt IE 8]><!--> <html class="no-js" lang="en" > <!--<![endif]--> |
| <head> |
| <meta charset="utf-8"> |
| |
| <meta name="viewport" content="width=device-width, initial-scale=1.0"> |
| |
| <title>apache_beam.typehints.schemas — Apache Beam 2.47.0 documentation</title> |
| |
| |
| |
| |
| |
| |
| |
| |
| <script type="text/javascript" src="../../../_static/js/modernizr.min.js"></script> |
| |
| |
| <script type="text/javascript" id="documentation_options" data-url_root="../../../" src="../../../_static/documentation_options.js"></script> |
| <script type="text/javascript" src="../../../_static/jquery.js"></script> |
| <script type="text/javascript" src="../../../_static/underscore.js"></script> |
| <script type="text/javascript" src="../../../_static/doctools.js"></script> |
| <script type="text/javascript" src="../../../_static/language_data.js"></script> |
| <script async="async" type="text/javascript" src="https://cdnjs.cloudflare.com/ajax/libs/mathjax/2.7.5/latest.js?config=TeX-AMS-MML_HTMLorMML"></script> |
| |
| <script type="text/javascript" src="../../../_static/js/theme.js"></script> |
| |
| |
| |
| |
| <link rel="stylesheet" href="../../../_static/css/theme.css" type="text/css" /> |
| <link rel="stylesheet" href="../../../_static/pygments.css" type="text/css" /> |
| <link rel="index" title="Index" href="../../../genindex.html" /> |
| <link rel="search" title="Search" href="../../../search.html" /> |
| </head> |
| |
| <body class="wy-body-for-nav"> |
| |
| |
| <div class="wy-grid-for-nav"> |
| |
| <nav data-toggle="wy-nav-shift" class="wy-nav-side"> |
| <div class="wy-side-scroll"> |
| <div class="wy-side-nav-search" > |
| |
| |
| |
| <a href="../../../index.html" class="icon icon-home"> Apache Beam |
| |
| |
| |
| </a> |
| |
| |
| |
| |
| <div class="version"> |
| 2.47.0 |
| </div> |
| |
| |
| |
| |
| <div role="search"> |
| <form id="rtd-search-form" class="wy-form" action="../../../search.html" method="get"> |
| <input type="text" name="q" placeholder="Search docs" /> |
| <input type="hidden" name="check_keywords" value="yes" /> |
| <input type="hidden" name="area" value="default" /> |
| </form> |
| </div> |
| |
| |
| </div> |
| |
| <div class="wy-menu wy-menu-vertical" data-spy="affix" role="navigation" aria-label="main navigation"> |
| |
| |
| |
| |
| |
| |
| <ul> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.coders.html">apache_beam.coders package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.dataframe.html">apache_beam.dataframe package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.io.html">apache_beam.io package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.metrics.html">apache_beam.metrics package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.ml.html">apache_beam.ml package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.options.html">apache_beam.options package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.portability.html">apache_beam.portability package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.runners.html">apache_beam.runners package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.testing.html">apache_beam.testing package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.transforms.html">apache_beam.transforms package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.typehints.html">apache_beam.typehints package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.utils.html">apache_beam.utils package</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.yaml.html">apache_beam.yaml package</a></li> |
| </ul> |
| <ul> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.error.html">apache_beam.error module</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.pipeline.html">apache_beam.pipeline module</a></li> |
| <li class="toctree-l1"><a class="reference internal" href="../../../apache_beam.pvalue.html">apache_beam.pvalue module</a></li> |
| </ul> |
| |
| |
| |
| </div> |
| </div> |
| </nav> |
| |
| <section data-toggle="wy-nav-shift" class="wy-nav-content-wrap"> |
| |
| |
| <nav class="wy-nav-top" aria-label="top navigation"> |
| |
| <i data-toggle="wy-nav-top" class="fa fa-bars"></i> |
| <a href="../../../index.html">Apache Beam</a> |
| |
| </nav> |
| |
| |
| <div class="wy-nav-content"> |
| |
| <div class="rst-content"> |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| <div role="navigation" aria-label="breadcrumbs navigation"> |
| |
| <ul class="wy-breadcrumbs"> |
| |
| <li><a href="../../../index.html">Docs</a> »</li> |
| |
| <li><a href="../../index.html">Module code</a> »</li> |
| |
| <li>apache_beam.typehints.schemas</li> |
| |
| |
| <li class="wy-breadcrumbs-aside"> |
| |
| </li> |
| |
| </ul> |
| |
| |
| <hr/> |
| </div> |
| <div role="main" class="document" itemscope="itemscope" itemtype="http://schema.org/Article"> |
| <div itemprop="articleBody"> |
| |
| <h1>Source code for apache_beam.typehints.schemas</h1><div class="highlight"><pre> |
| <span></span><span class="c1">#</span> |
| <span class="c1"># Licensed to the Apache Software Foundation (ASF) under one or more</span> |
| <span class="c1"># contributor license agreements. See the NOTICE file distributed with</span> |
| <span class="c1"># this work for additional information regarding copyright ownership.</span> |
| <span class="c1"># The ASF licenses this file to You under the Apache License, Version 2.0</span> |
| <span class="c1"># (the "License"); you may not use this file except in compliance with</span> |
| <span class="c1"># the License. You may obtain a copy of the License at</span> |
| <span class="c1">#</span> |
| <span class="c1"># http://www.apache.org/licenses/LICENSE-2.0</span> |
| <span class="c1">#</span> |
| <span class="c1"># Unless required by applicable law or agreed to in writing, software</span> |
| <span class="c1"># distributed under the License is distributed on an "AS IS" BASIS,</span> |
| <span class="c1"># WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.</span> |
| <span class="c1"># See the License for the specific language governing permissions and</span> |
| <span class="c1"># limitations under the License.</span> |
| <span class="c1">#</span> |
| |
| <span class="sd">""" Support for mapping python types to proto Schemas and back again.</span> |
| |
| <span class="sd">Imposes a mapping between common Python types and Beam portable schemas</span> |
| <span class="sd">(https://s.apache.org/beam-schemas)::</span> |
| |
| <span class="sd"> Python Schema</span> |
| <span class="sd"> np.int8 <-----> BYTE</span> |
| <span class="sd"> np.int16 <-----> INT16</span> |
| <span class="sd"> np.int32 <-----> INT32</span> |
| <span class="sd"> np.int64 <-----> INT64</span> |
| <span class="sd"> int ------> INT64</span> |
| <span class="sd"> np.float32 <-----> FLOAT</span> |
| <span class="sd"> np.float64 <-----> DOUBLE</span> |
| <span class="sd"> float ------> DOUBLE</span> |
| <span class="sd"> bool <-----> BOOLEAN</span> |
| <span class="sd"> str <-----> STRING</span> |
| <span class="sd"> bytes <-----> BYTES</span> |
| <span class="sd"> ByteString ------> BYTES</span> |
| <span class="sd"> Timestamp <-----> LogicalType(urn="beam:logical_type:micros_instant:v1")</span> |
| <span class="sd"> Decimal <-----> LogicalType(urn="beam:logical_type:fixed_decimal:v1")</span> |
| <span class="sd"> Mapping <-----> MapType</span> |
| <span class="sd"> Sequence <-----> ArrayType</span> |
| <span class="sd"> NamedTuple <-----> RowType</span> |
| <span class="sd"> beam.Row ------> RowType</span> |
| |
| <span class="sd">One direction mapping of Python types from Beam portable schemas:</span> |
| |
| <span class="sd"> bytes</span> |
| <span class="sd"> <------ LogicalType(urn="beam:logical_type:fixed_bytes:v1")</span> |
| <span class="sd"> <------ LogicalType(urn="beam:logical_type:var_bytes:v1")</span> |
| <span class="sd"> str</span> |
| <span class="sd"> <------ LogicalType(urn="beam:logical_type:fixed_char:v1")</span> |
| <span class="sd"> <------ LogicalType(urn="beam:logical_type:var_char:v1")</span> |
| <span class="sd"> Timestamp</span> |
| <span class="sd"> <------ LogicalType(urn="beam:logical_type:millis_instant:v1")</span> |
| |
| <span class="sd">Note that some of these mappings are provided as conveniences,</span> |
| <span class="sd">but they are lossy and will not survive a roundtrip from python to Beam schemas</span> |
| <span class="sd">and back. For example, the Python type :code:`int` will map to :code:`INT64` in</span> |
| <span class="sd">Beam schemas but converting that back to a Python type will yield</span> |
| <span class="sd">:code:`np.int64`.</span> |
| |
| <span class="sd">:code:`nullable=True` on a Beam :code:`FieldType` is represented in Python by</span> |
| <span class="sd">wrapping the type in :code:`Optional`.</span> |
| |
| <span class="sd">This module is intended for internal use only. Nothing defined here provides</span> |
| <span class="sd">any backwards-compatibility guarantee.</span> |
| <span class="sd">"""</span> |
| |
| <span class="c1"># pytype: skip-file</span> |
| |
| <span class="kn">import</span> <span class="nn">decimal</span> |
| <span class="kn">import</span> <span class="nn">logging</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Any</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">ByteString</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Dict</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Generic</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">List</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Mapping</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">NamedTuple</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Optional</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Sequence</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Tuple</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">TypeVar</span> |
| <span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Union</span> |
| |
| <span class="kn">import</span> <span class="nn">numpy</span> <span class="k">as</span> <span class="nn">np</span> |
| <span class="kn">from</span> <span class="nn">google.protobuf</span> <span class="kn">import</span> <span class="n">text_format</span> |
| |
| <span class="kn">from</span> <span class="nn">apache_beam.portability</span> <span class="kn">import</span> <span class="n">common_urns</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.portability.api</span> <span class="kn">import</span> <span class="n">schema_pb2</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints</span> <span class="kn">import</span> <span class="n">row_type</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints.native_type_compatibility</span> <span class="kn">import</span> <span class="n">_get_args</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints.native_type_compatibility</span> <span class="kn">import</span> <span class="n">_match_is_exactly_mapping</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints.native_type_compatibility</span> <span class="kn">import</span> <span class="n">_match_is_optional</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints.native_type_compatibility</span> <span class="kn">import</span> <span class="n">_safe_issubclass</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints.native_type_compatibility</span> <span class="kn">import</span> <span class="n">extract_optional_type</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints.native_type_compatibility</span> <span class="kn">import</span> <span class="n">match_is_named_tuple</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints.schema_registry</span> <span class="kn">import</span> <span class="n">SCHEMA_REGISTRY</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.typehints.schema_registry</span> <span class="kn">import</span> <span class="n">SchemaTypeRegistry</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.utils</span> <span class="kn">import</span> <span class="n">proto_utils</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.utils.python_callable</span> <span class="kn">import</span> <span class="n">PythonCallableWithSource</span> |
| <span class="kn">from</span> <span class="nn">apache_beam.utils.timestamp</span> <span class="kn">import</span> <span class="n">Timestamp</span> |
| |
| <span class="n">PYTHON_ANY_URN</span> <span class="o">=</span> <span class="s2">"beam:logical:pythonsdk_any:v1"</span> |
| |
| <span class="c1"># Bi-directional mappings</span> |
| <span class="n">_PRIMITIVES</span> <span class="o">=</span> <span class="p">(</span> |
| <span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">int8</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BYTE</span><span class="p">),</span> |
| <span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">int16</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT16</span><span class="p">),</span> |
| <span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT32</span><span class="p">),</span> |
| <span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">int64</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT64</span><span class="p">),</span> |
| <span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">float32</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FLOAT</span><span class="p">),</span> |
| <span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">float64</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">DOUBLE</span><span class="p">),</span> |
| <span class="p">(</span><span class="nb">str</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">STRING</span><span class="p">),</span> |
| <span class="p">(</span><span class="nb">bool</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BOOLEAN</span><span class="p">),</span> |
| <span class="p">(</span><span class="nb">bytes</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BYTES</span><span class="p">),</span> |
| <span class="p">)</span> |
| |
| <span class="n">PRIMITIVE_TO_ATOMIC_TYPE</span> <span class="o">=</span> <span class="nb">dict</span><span class="p">((</span><span class="n">typ</span><span class="p">,</span> <span class="n">atomic</span><span class="p">)</span> <span class="k">for</span> <span class="n">typ</span><span class="p">,</span> <span class="n">atomic</span> <span class="ow">in</span> <span class="n">_PRIMITIVES</span><span class="p">)</span> |
| <span class="n">ATOMIC_TYPE_TO_PRIMITIVE</span> <span class="o">=</span> <span class="nb">dict</span><span class="p">((</span><span class="n">atomic</span><span class="p">,</span> <span class="n">typ</span><span class="p">)</span> <span class="k">for</span> <span class="n">typ</span><span class="p">,</span> <span class="n">atomic</span> <span class="ow">in</span> <span class="n">_PRIMITIVES</span><span class="p">)</span> |
| |
| <span class="c1"># One-way mappings</span> |
| <span class="n">PRIMITIVE_TO_ATOMIC_TYPE</span><span class="o">.</span><span class="n">update</span><span class="p">({</span> |
| <span class="n">ByteString</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BYTES</span><span class="p">,</span> |
| <span class="c1"># Allow users to specify a native int, and use INT64 as the cross-language</span> |
| <span class="c1"># representation. Technically ints have unlimited precision, but RowCoder</span> |
| <span class="c1"># should throw an error if it sees one with a bit width > 64 when encoding.</span> |
| <span class="nb">int</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT64</span><span class="p">,</span> |
| <span class="nb">float</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">DOUBLE</span><span class="p">,</span> |
| <span class="p">})</span> |
| |
| <span class="n">_LOGGER</span> <span class="o">=</span> <span class="n">logging</span><span class="o">.</span><span class="n">getLogger</span><span class="p">(</span><span class="vm">__name__</span><span class="p">)</span> |
| |
| <span class="c1"># Serialized schema_pb2.Schema w/o id to id.</span> |
| <span class="n">_SCHEMA_ID_CACHE</span> <span class="o">=</span> <span class="p">{}</span> |
| |
| |
| <div class="viewcode-block" id="named_fields_to_schema"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.named_fields_to_schema">[docs]</a><span class="k">def</span> <span class="nf">named_fields_to_schema</span><span class="p">(</span> |
| <span class="n">names_and_types</span><span class="p">:</span> <span class="n">Union</span><span class="p">[</span><span class="n">Dict</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="nb">type</span><span class="p">],</span> <span class="n">Sequence</span><span class="p">[</span><span class="n">Tuple</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="nb">type</span><span class="p">]]],</span> |
| <span class="n">schema_id</span><span class="p">:</span> <span class="n">Optional</span><span class="p">[</span><span class="nb">str</span><span class="p">]</span> <span class="o">=</span> <span class="kc">None</span><span class="p">,</span> |
| <span class="n">schema_options</span><span class="p">:</span> <span class="n">Optional</span><span class="p">[</span><span class="n">Sequence</span><span class="p">[</span><span class="n">Tuple</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">Any</span><span class="p">]]]</span> <span class="o">=</span> <span class="kc">None</span><span class="p">,</span> |
| <span class="n">field_options</span><span class="p">:</span> <span class="n">Optional</span><span class="p">[</span><span class="n">Dict</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">Sequence</span><span class="p">[</span><span class="n">Tuple</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">Any</span><span class="p">]]]]</span> <span class="o">=</span> <span class="kc">None</span><span class="p">,</span> |
| <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span><span class="p">,</span> |
| <span class="p">):</span> |
| <span class="n">schema_options</span> <span class="o">=</span> <span class="n">schema_options</span> <span class="ow">or</span> <span class="p">[]</span> |
| <span class="n">field_options</span> <span class="o">=</span> <span class="n">field_options</span> <span class="ow">or</span> <span class="p">{}</span> |
| |
| <span class="k">if</span> <span class="nb">isinstance</span><span class="p">(</span><span class="n">names_and_types</span><span class="p">,</span> <span class="nb">dict</span><span class="p">):</span> |
| <span class="n">names_and_types</span> <span class="o">=</span> <span class="n">names_and_types</span><span class="o">.</span><span class="n">items</span><span class="p">()</span> |
| |
| <span class="n">schema</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Schema</span><span class="p">(</span> |
| <span class="n">fields</span><span class="o">=</span><span class="p">[</span> |
| <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Field</span><span class="p">(</span> |
| <span class="n">name</span><span class="o">=</span><span class="n">name</span><span class="p">,</span> |
| <span class="nb">type</span><span class="o">=</span><span class="n">typing_to_runner_api</span><span class="p">(</span><span class="nb">type</span><span class="p">),</span> |
| <span class="n">options</span><span class="o">=</span><span class="p">[</span> |
| <span class="n">option_to_runner_api</span><span class="p">(</span><span class="n">option_tuple</span><span class="p">)</span> |
| <span class="k">for</span> <span class="n">option_tuple</span> <span class="ow">in</span> <span class="n">field_options</span><span class="o">.</span><span class="n">get</span><span class="p">(</span><span class="n">name</span><span class="p">,</span> <span class="p">[])</span> |
| <span class="p">],</span> |
| <span class="p">)</span> <span class="k">for</span> <span class="p">(</span><span class="n">name</span><span class="p">,</span> <span class="nb">type</span><span class="p">)</span> <span class="ow">in</span> <span class="n">names_and_types</span> |
| <span class="p">],</span> |
| <span class="n">options</span><span class="o">=</span><span class="p">[</span> |
| <span class="n">option_to_runner_api</span><span class="p">(</span><span class="n">option_tuple</span><span class="p">)</span> <span class="k">for</span> <span class="n">option_tuple</span> <span class="ow">in</span> <span class="n">schema_options</span> |
| <span class="p">])</span> |
| |
| <span class="k">if</span> <span class="n">schema_id</span> <span class="ow">is</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="n">key</span> <span class="o">=</span> <span class="n">schema</span><span class="o">.</span><span class="n">SerializeToString</span><span class="p">()</span> |
| <span class="k">if</span> <span class="n">key</span> <span class="ow">not</span> <span class="ow">in</span> <span class="n">_SCHEMA_ID_CACHE</span><span class="p">:</span> |
| <span class="n">_SCHEMA_ID_CACHE</span><span class="p">[</span><span class="n">key</span><span class="p">]</span> <span class="o">=</span> <span class="n">schema_registry</span><span class="o">.</span><span class="n">generate_new_id</span><span class="p">()</span> |
| <span class="n">schema_id</span> <span class="o">=</span> <span class="n">_SCHEMA_ID_CACHE</span><span class="p">[</span><span class="n">key</span><span class="p">]</span> |
| |
| <span class="n">schema</span><span class="o">.</span><span class="n">id</span> <span class="o">=</span> <span class="n">schema_id</span> |
| <span class="k">return</span> <span class="n">schema</span></div> |
| |
| |
| <div class="viewcode-block" id="named_fields_from_schema"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.named_fields_from_schema">[docs]</a><span class="k">def</span> <span class="nf">named_fields_from_schema</span><span class="p">(</span> |
| <span class="n">schema</span><span class="p">):</span> <span class="c1"># (schema_pb2.Schema) -> typing.List[typing.Tuple[str, type]]</span> |
| <span class="k">return</span> <span class="p">[(</span><span class="n">field</span><span class="o">.</span><span class="n">name</span><span class="p">,</span> <span class="n">typing_from_runner_api</span><span class="p">(</span><span class="n">field</span><span class="o">.</span><span class="n">type</span><span class="p">))</span> |
| <span class="k">for</span> <span class="n">field</span> <span class="ow">in</span> <span class="n">schema</span><span class="o">.</span><span class="n">fields</span><span class="p">]</span></div> |
| |
| |
| <div class="viewcode-block" id="typing_to_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.typing_to_runner_api">[docs]</a><span class="k">def</span> <span class="nf">typing_to_runner_api</span><span class="p">(</span> |
| <span class="n">type_</span><span class="p">:</span> <span class="nb">type</span><span class="p">,</span> |
| <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span> |
| <span class="p">)</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">SchemaTranslation</span><span class="p">(</span> |
| <span class="n">schema_registry</span><span class="o">=</span><span class="n">schema_registry</span><span class="p">)</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">(</span><span class="n">type_</span><span class="p">)</span></div> |
| |
| |
| <div class="viewcode-block" id="typing_from_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.typing_from_runner_api">[docs]</a><span class="k">def</span> <span class="nf">typing_from_runner_api</span><span class="p">(</span> |
| <span class="n">fieldtype_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">,</span> |
| <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span><span class="p">)</span> <span class="o">-></span> <span class="nb">type</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">SchemaTranslation</span><span class="p">(</span> |
| <span class="n">schema_registry</span><span class="o">=</span><span class="n">schema_registry</span><span class="p">)</span><span class="o">.</span><span class="n">typing_from_runner_api</span><span class="p">(</span><span class="n">fieldtype_proto</span><span class="p">)</span></div> |
| |
| |
| <div class="viewcode-block" id="value_to_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.value_to_runner_api">[docs]</a><span class="k">def</span> <span class="nf">value_to_runner_api</span><span class="p">(</span> |
| <span class="n">type_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">,</span> |
| <span class="n">value</span><span class="p">,</span> |
| <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span> |
| <span class="p">)</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldValue</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">SchemaTranslation</span><span class="p">(</span><span class="n">schema_registry</span><span class="o">=</span><span class="n">schema_registry</span><span class="p">)</span><span class="o">.</span><span class="n">value_to_runner_api</span><span class="p">(</span> |
| <span class="n">type_proto</span><span class="p">,</span> <span class="n">value</span><span class="p">)</span></div> |
| |
| |
| <div class="viewcode-block" id="value_from_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.value_from_runner_api">[docs]</a><span class="k">def</span> <span class="nf">value_from_runner_api</span><span class="p">(</span> |
| <span class="n">type_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">,</span> |
| <span class="n">value_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldValue</span><span class="p">,</span> |
| <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span> |
| <span class="p">)</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldValue</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">SchemaTranslation</span><span class="p">(</span> |
| <span class="n">schema_registry</span><span class="o">=</span><span class="n">schema_registry</span><span class="p">)</span><span class="o">.</span><span class="n">value_from_runner_api</span><span class="p">(</span> |
| <span class="n">type_proto</span><span class="p">,</span> <span class="n">value_proto</span><span class="p">)</span></div> |
| |
| |
| <div class="viewcode-block" id="option_to_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.option_to_runner_api">[docs]</a><span class="k">def</span> <span class="nf">option_to_runner_api</span><span class="p">(</span> |
| <span class="n">option</span><span class="p">:</span> <span class="n">Tuple</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">Any</span><span class="p">],</span> |
| <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span><span class="p">)</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Option</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">SchemaTranslation</span><span class="p">(</span> |
| <span class="n">schema_registry</span><span class="o">=</span><span class="n">schema_registry</span><span class="p">)</span><span class="o">.</span><span class="n">option_to_runner_api</span><span class="p">(</span><span class="n">option</span><span class="p">)</span></div> |
| |
| |
| <div class="viewcode-block" id="option_from_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.option_from_runner_api">[docs]</a><span class="k">def</span> <span class="nf">option_from_runner_api</span><span class="p">(</span> |
| <span class="n">option_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Option</span><span class="p">,</span> |
| <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span><span class="p">)</span> <span class="o">-></span> <span class="n">Tuple</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">Any</span><span class="p">]:</span> |
| <span class="k">return</span> <span class="n">SchemaTranslation</span><span class="p">(</span> |
| <span class="n">schema_registry</span><span class="o">=</span><span class="n">schema_registry</span><span class="p">)</span><span class="o">.</span><span class="n">option_from_runner_api</span><span class="p">(</span><span class="n">option_proto</span><span class="p">)</span></div> |
| |
| |
| <div class="viewcode-block" id="SchemaTranslation"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation">[docs]</a><span class="k">class</span> <span class="nc">SchemaTranslation</span><span class="p">(</span><span class="nb">object</span><span class="p">):</span> |
| <span class="k">def</span> <span class="fm">__init__</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span><span class="p">):</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">schema_registry</span> <span class="o">=</span> <span class="n">schema_registry</span> |
| |
| <div class="viewcode-block" id="SchemaTranslation.typing_to_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.typing_to_runner_api">[docs]</a> <span class="k">def</span> <span class="nf">typing_to_runner_api</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">type_</span><span class="p">:</span> <span class="nb">type</span><span class="p">)</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">:</span> |
| <span class="k">if</span> <span class="nb">isinstance</span><span class="p">(</span><span class="n">type_</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Schema</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">(</span><span class="n">row_type</span><span class="o">=</span><span class="n">schema_pb2</span><span class="o">.</span><span class="n">RowType</span><span class="p">(</span><span class="n">schema</span><span class="o">=</span><span class="n">type_</span><span class="p">))</span> |
| |
| <span class="k">if</span> <span class="nb">isinstance</span><span class="p">(</span><span class="n">type_</span><span class="p">,</span> <span class="n">row_type</span><span class="o">.</span><span class="n">RowTypeConstraint</span><span class="p">):</span> |
| <span class="k">if</span> <span class="n">type_</span><span class="o">.</span><span class="n">schema_id</span> <span class="ow">is</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="n">schema_id</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span><span class="o">.</span><span class="n">generate_new_id</span><span class="p">()</span> |
| <span class="n">type_</span><span class="o">.</span><span class="n">set_schema_id</span><span class="p">(</span><span class="n">schema_id</span><span class="p">)</span> |
| <span class="n">schema</span> <span class="o">=</span> <span class="kc">None</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="n">schema_id</span> <span class="o">=</span> <span class="n">type_</span><span class="o">.</span><span class="n">schema_id</span> |
| <span class="n">schema</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">schema_registry</span><span class="o">.</span><span class="n">get_schema_by_id</span><span class="p">(</span><span class="n">schema_id</span><span class="p">)</span> |
| |
| <span class="k">if</span> <span class="n">schema</span> <span class="ow">is</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="c1"># Either user_type was not annotated with a schema id, or there was</span> |
| <span class="c1"># no schema in the registry with the id. The latter should only happen</span> |
| <span class="c1"># in tests.</span> |
| <span class="c1"># Either way, we need to generate a new schema proto.</span> |
| <span class="n">schema</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Schema</span><span class="p">(</span> |
| <span class="n">fields</span><span class="o">=</span><span class="p">[</span> |
| <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Field</span><span class="p">(</span> |
| <span class="n">name</span><span class="o">=</span><span class="n">field_name</span><span class="p">,</span> |
| <span class="nb">type</span><span class="o">=</span><span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">(</span><span class="n">field_type</span><span class="p">),</span> |
| <span class="n">options</span><span class="o">=</span><span class="p">[</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">option_to_runner_api</span><span class="p">(</span><span class="n">option_tuple</span><span class="p">)</span> |
| <span class="k">for</span> <span class="n">option_tuple</span> <span class="ow">in</span> <span class="n">type_</span><span class="o">.</span><span class="n">field_options</span><span class="p">(</span><span class="n">field_name</span><span class="p">)</span> |
| <span class="p">],</span> |
| <span class="p">)</span> <span class="k">for</span> <span class="p">(</span><span class="n">field_name</span><span class="p">,</span> <span class="n">field_type</span><span class="p">)</span> <span class="ow">in</span> <span class="n">type_</span><span class="o">.</span><span class="n">_fields</span> |
| <span class="p">],</span> |
| <span class="nb">id</span><span class="o">=</span><span class="n">schema_id</span><span class="p">,</span> |
| <span class="n">options</span><span class="o">=</span><span class="p">[</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">option_to_runner_api</span><span class="p">(</span><span class="n">option_tuple</span><span class="p">)</span> |
| <span class="k">for</span> <span class="n">option_tuple</span> <span class="ow">in</span> <span class="n">type_</span><span class="o">.</span><span class="n">schema_options</span> |
| <span class="p">],</span> |
| <span class="p">)</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">schema_registry</span><span class="o">.</span><span class="n">add</span><span class="p">(</span><span class="n">type_</span><span class="o">.</span><span class="n">user_type</span><span class="p">,</span> <span class="n">schema</span><span class="p">)</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">(</span><span class="n">row_type</span><span class="o">=</span><span class="n">schema_pb2</span><span class="o">.</span><span class="n">RowType</span><span class="p">(</span><span class="n">schema</span><span class="o">=</span><span class="n">schema</span><span class="p">))</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="c1"># See if this is coercible to a RowTypeConstraint (e.g. a NamedTuple or</span> |
| <span class="c1"># dataclass)</span> |
| <span class="n">row_type_constraint</span> <span class="o">=</span> <span class="n">row_type</span><span class="o">.</span><span class="n">RowTypeConstraint</span><span class="o">.</span><span class="n">from_user_type</span><span class="p">(</span><span class="n">type_</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">row_type_constraint</span> <span class="ow">is</span> <span class="ow">not</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="k">return</span> <span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">(</span><span class="n">row_type_constraint</span><span class="p">)</span> |
| |
| <span class="c1"># All concrete types (other than NamedTuple sub-classes) should map to</span> |
| <span class="c1"># a supported primitive type.</span> |
| <span class="k">if</span> <span class="n">type_</span> <span class="ow">in</span> <span class="n">PRIMITIVE_TO_ATOMIC_TYPE</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">(</span><span class="n">atomic_type</span><span class="o">=</span><span class="n">PRIMITIVE_TO_ATOMIC_TYPE</span><span class="p">[</span><span class="n">type_</span><span class="p">])</span> |
| |
| <span class="k">elif</span> <span class="n">_match_is_exactly_mapping</span><span class="p">(</span><span class="n">type_</span><span class="p">):</span> |
| <span class="n">key_type</span><span class="p">,</span> <span class="n">value_type</span> <span class="o">=</span> <span class="nb">map</span><span class="p">(</span><span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">,</span> <span class="n">_get_args</span><span class="p">(</span><span class="n">type_</span><span class="p">))</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">(</span> |
| <span class="n">map_type</span><span class="o">=</span><span class="n">schema_pb2</span><span class="o">.</span><span class="n">MapType</span><span class="p">(</span><span class="n">key_type</span><span class="o">=</span><span class="n">key_type</span><span class="p">,</span> <span class="n">value_type</span><span class="o">=</span><span class="n">value_type</span><span class="p">))</span> |
| |
| <span class="k">elif</span> <span class="n">_match_is_optional</span><span class="p">(</span><span class="n">type_</span><span class="p">):</span> |
| <span class="c1"># It's possible that a user passes us Optional[Optional[T]], but in python</span> |
| <span class="c1"># typing this is indistinguishable from Optional[T] - both resolve to</span> |
| <span class="c1"># Union[T, None] - so there's no need to check for that case here.</span> |
| <span class="n">result</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">(</span><span class="n">extract_optional_type</span><span class="p">(</span><span class="n">type_</span><span class="p">))</span> |
| <span class="n">result</span><span class="o">.</span><span class="n">nullable</span> <span class="o">=</span> <span class="kc">True</span> |
| <span class="k">return</span> <span class="n">result</span> |
| |
| <span class="k">elif</span> <span class="n">_safe_issubclass</span><span class="p">(</span><span class="n">type_</span><span class="p">,</span> <span class="n">Sequence</span><span class="p">):</span> |
| <span class="n">element_type</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">(</span><span class="n">_get_args</span><span class="p">(</span><span class="n">type_</span><span class="p">)[</span><span class="mi">0</span><span class="p">])</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">(</span> |
| <span class="n">array_type</span><span class="o">=</span><span class="n">schema_pb2</span><span class="o">.</span><span class="n">ArrayType</span><span class="p">(</span><span class="n">element_type</span><span class="o">=</span><span class="n">element_type</span><span class="p">))</span> |
| |
| <span class="k">elif</span> <span class="n">_safe_issubclass</span><span class="p">(</span><span class="n">type_</span><span class="p">,</span> <span class="n">Mapping</span><span class="p">):</span> |
| <span class="n">key_type</span><span class="p">,</span> <span class="n">value_type</span> <span class="o">=</span> <span class="nb">map</span><span class="p">(</span><span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">,</span> <span class="n">_get_args</span><span class="p">(</span><span class="n">type_</span><span class="p">))</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">(</span> |
| <span class="n">map_type</span><span class="o">=</span><span class="n">schema_pb2</span><span class="o">.</span><span class="n">MapType</span><span class="p">(</span><span class="n">key_type</span><span class="o">=</span><span class="n">key_type</span><span class="p">,</span> <span class="n">value_type</span><span class="o">=</span><span class="n">value_type</span><span class="p">))</span> |
| |
| <span class="k">try</span><span class="p">:</span> |
| <span class="n">logical_type</span> <span class="o">=</span> <span class="n">LogicalType</span><span class="o">.</span><span class="n">from_typing</span><span class="p">(</span><span class="n">type_</span><span class="p">)</span> |
| <span class="k">except</span> <span class="ne">ValueError</span><span class="p">:</span> |
| <span class="c1"># Unknown type, just treat it like Any</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">(</span> |
| <span class="n">logical_type</span><span class="o">=</span><span class="n">schema_pb2</span><span class="o">.</span><span class="n">LogicalType</span><span class="p">(</span><span class="n">urn</span><span class="o">=</span><span class="n">PYTHON_ANY_URN</span><span class="p">),</span> |
| <span class="n">nullable</span><span class="o">=</span><span class="kc">True</span><span class="p">)</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="n">argument_type</span> <span class="o">=</span> <span class="kc">None</span> |
| <span class="n">argument</span> <span class="o">=</span> <span class="kc">None</span> |
| <span class="k">if</span> <span class="n">logical_type</span><span class="o">.</span><span class="n">argument_type</span><span class="p">()</span> <span class="ow">is</span> <span class="ow">not</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="n">argument_type</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">(</span><span class="n">logical_type</span><span class="o">.</span><span class="n">argument_type</span><span class="p">())</span> |
| <span class="k">try</span><span class="p">:</span> |
| <span class="n">argument</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">value_to_runner_api</span><span class="p">(</span> |
| <span class="n">argument_type</span><span class="p">,</span> <span class="n">logical_type</span><span class="o">.</span><span class="n">argument</span><span class="p">())</span> |
| <span class="k">except</span> <span class="ne">ValueError</span><span class="p">:</span> |
| <span class="c1"># TODO(https://github.com/apache/beam/issues/23373): Complete support</span> |
| <span class="c1"># for logical types that require arguments beyond atomic type.</span> |
| <span class="c1"># For now, skip arguments.</span> |
| <span class="n">argument</span> <span class="o">=</span> <span class="kc">None</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">(</span> |
| <span class="n">logical_type</span><span class="o">=</span><span class="n">schema_pb2</span><span class="o">.</span><span class="n">LogicalType</span><span class="p">(</span> |
| <span class="n">urn</span><span class="o">=</span><span class="n">logical_type</span><span class="o">.</span><span class="n">urn</span><span class="p">(),</span> |
| <span class="n">representation</span><span class="o">=</span><span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">(</span> |
| <span class="n">logical_type</span><span class="o">.</span><span class="n">representation_type</span><span class="p">()),</span> |
| <span class="n">argument_type</span><span class="o">=</span><span class="n">argument_type</span><span class="p">,</span> |
| <span class="n">argument</span><span class="o">=</span><span class="n">argument</span><span class="p">))</span></div> |
| |
| <div class="viewcode-block" id="SchemaTranslation.atomic_value_from_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.atomic_value_from_runner_api">[docs]</a> <span class="k">def</span> <span class="nf">atomic_value_from_runner_api</span><span class="p">(</span> |
| <span class="bp">self</span><span class="p">,</span> |
| <span class="n">atomic_type</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicType</span><span class="p">,</span> |
| <span class="n">atomic_value</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">):</span> |
| <span class="k">if</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BYTE</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">np</span><span class="o">.</span><span class="n">int8</span><span class="p">(</span><span class="n">atomic_value</span><span class="o">.</span><span class="n">byte</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT16</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">np</span><span class="o">.</span><span class="n">int16</span><span class="p">(</span><span class="n">atomic_value</span><span class="o">.</span><span class="n">int16</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT32</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">(</span><span class="n">atomic_value</span><span class="o">.</span><span class="n">int32</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT64</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">np</span><span class="o">.</span><span class="n">int64</span><span class="p">(</span><span class="n">atomic_value</span><span class="o">.</span><span class="n">int64</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FLOAT</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">np</span><span class="o">.</span><span class="n">float32</span><span class="p">(</span><span class="n">atomic_value</span><span class="o">.</span><span class="n">float</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">DOUBLE</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">np</span><span class="o">.</span><span class="n">float64</span><span class="p">(</span><span class="n">atomic_value</span><span class="o">.</span><span class="n">double</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">STRING</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">atomic_value</span><span class="o">.</span><span class="n">string</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BOOLEAN</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">atomic_value</span><span class="o">.</span><span class="n">boolean</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BYTES</span><span class="p">:</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">atomic_value</span><span class="o">.</span><span class="n">bytes</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="sa">f</span><span class="s2">"Unrecognized atomic_type (</span><span class="si">{</span><span class="n">atomic_type</span><span class="si">}</span><span class="s2">) "</span> |
| <span class="sa">f</span><span class="s2">"when decoding value </span><span class="si">{</span><span class="n">atomic_value</span><span class="si">!r}</span><span class="s2">"</span><span class="p">)</span> |
| |
| <span class="k">return</span> <span class="n">value</span></div> |
| |
| <div class="viewcode-block" id="SchemaTranslation.atomic_value_to_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.atomic_value_to_runner_api">[docs]</a> <span class="k">def</span> <span class="nf">atomic_value_to_runner_api</span><span class="p">(</span> |
| <span class="bp">self</span><span class="p">,</span> <span class="n">atomic_type</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicType</span><span class="p">,</span> |
| <span class="n">value</span><span class="p">)</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">:</span> |
| <span class="k">if</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BYTE</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="n">byte</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT16</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="n">int16</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT32</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="n">int32</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">INT64</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="n">int64</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FLOAT</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="nb">float</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">DOUBLE</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="n">double</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">STRING</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="n">string</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BOOLEAN</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="n">boolean</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">atomic_type</span> <span class="o">==</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">BYTES</span><span class="p">:</span> |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">AtomicTypeValue</span><span class="p">(</span><span class="nb">bytes</span><span class="o">=</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"Unrecognized atomic_type </span><span class="si">{atomic_type}</span><span class="s2"> when encoding value </span><span class="si">{value}</span><span class="s2">"</span><span class="p">)</span> |
| |
| <span class="k">return</span> <span class="n">atomic_value</span></div> |
| |
| <div class="viewcode-block" id="SchemaTranslation.value_from_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.value_from_runner_api">[docs]</a> <span class="k">def</span> <span class="nf">value_from_runner_api</span><span class="p">(</span> |
| <span class="bp">self</span><span class="p">,</span> |
| <span class="n">type_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">,</span> |
| <span class="n">value_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldValue</span><span class="p">):</span> |
| <span class="k">if</span> <span class="n">type_proto</span><span class="o">.</span><span class="n">WhichOneof</span><span class="p">(</span><span class="s2">"type_info"</span><span class="p">)</span> <span class="o">!=</span> <span class="s2">"atomic_type"</span><span class="p">:</span> |
| <span class="c1"># TODO: Allow other value types</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"Encounterd option with unsupported type. Only "</span> |
| <span class="sa">f</span><span class="s2">"atomic_type options are supported: </span><span class="si">{</span><span class="n">type_proto</span><span class="si">}</span><span class="s2">"</span><span class="p">)</span> |
| |
| <span class="n">value</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">atomic_value_from_runner_api</span><span class="p">(</span> |
| <span class="n">type_proto</span><span class="o">.</span><span class="n">atomic_type</span><span class="p">,</span> <span class="n">value_proto</span><span class="o">.</span><span class="n">atomic_value</span><span class="p">)</span> |
| <span class="k">return</span> <span class="n">value</span></div> |
| |
| <div class="viewcode-block" id="SchemaTranslation.value_to_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.value_to_runner_api">[docs]</a> <span class="k">def</span> <span class="nf">value_to_runner_api</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">typing_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="k">if</span> <span class="n">typing_proto</span><span class="o">.</span><span class="n">WhichOneof</span><span class="p">(</span><span class="s2">"type_info"</span><span class="p">)</span> <span class="o">!=</span> <span class="s2">"atomic_type"</span><span class="p">:</span> |
| <span class="c1"># TODO: Allow other value types</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"Only atomic_type option values are currently supported in Python. "</span> |
| <span class="sa">f</span><span class="s2">"Got </span><span class="si">{</span><span class="n">value</span><span class="si">!r}</span><span class="s2">, which maps to fieldtype </span><span class="si">{</span><span class="n">typing_proto</span><span class="si">!r}</span><span class="s2">."</span><span class="p">)</span> |
| |
| <span class="n">atomic_value</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">atomic_value_to_runner_api</span><span class="p">(</span> |
| <span class="n">typing_proto</span><span class="o">.</span><span class="n">atomic_type</span><span class="p">,</span> <span class="n">value</span><span class="p">)</span> |
| <span class="n">value_proto</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldValue</span><span class="p">(</span><span class="n">atomic_value</span><span class="o">=</span><span class="n">atomic_value</span><span class="p">)</span> |
| <span class="k">return</span> <span class="n">value_proto</span></div> |
| |
| <div class="viewcode-block" id="SchemaTranslation.option_from_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.option_from_runner_api">[docs]</a> <span class="k">def</span> <span class="nf">option_from_runner_api</span><span class="p">(</span> |
| <span class="bp">self</span><span class="p">,</span> <span class="n">option_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Option</span><span class="p">)</span> <span class="o">-></span> <span class="n">Tuple</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">Any</span><span class="p">]:</span> |
| <span class="k">if</span> <span class="ow">not</span> <span class="n">option_proto</span><span class="o">.</span><span class="n">HasField</span><span class="p">(</span><span class="s1">'type'</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">option_proto</span><span class="o">.</span><span class="n">name</span><span class="p">,</span> <span class="kc">None</span> |
| |
| <span class="n">value</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">value_from_runner_api</span><span class="p">(</span><span class="n">option_proto</span><span class="o">.</span><span class="n">type</span><span class="p">,</span> <span class="n">option_proto</span><span class="o">.</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">return</span> <span class="n">option_proto</span><span class="o">.</span><span class="n">name</span><span class="p">,</span> <span class="n">value</span></div> |
| |
| <div class="viewcode-block" id="SchemaTranslation.option_to_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.option_to_runner_api">[docs]</a> <span class="k">def</span> <span class="nf">option_to_runner_api</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">option</span><span class="p">:</span> <span class="n">Tuple</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">Any</span><span class="p">])</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Option</span><span class="p">:</span> |
| <span class="n">name</span><span class="p">,</span> <span class="n">value</span> <span class="o">=</span> <span class="n">option</span> |
| |
| <span class="k">if</span> <span class="n">value</span> <span class="ow">is</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="c1"># a value of None indicates the option is just a flag.</span> |
| <span class="c1"># Don't set type, value</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Option</span><span class="p">(</span><span class="n">name</span><span class="o">=</span><span class="n">name</span><span class="p">)</span> |
| |
| <span class="n">type_proto</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">typing_to_runner_api</span><span class="p">(</span><span class="nb">type</span><span class="p">(</span><span class="n">value</span><span class="p">))</span> |
| <span class="n">value_proto</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">value_to_runner_api</span><span class="p">(</span><span class="n">type_proto</span><span class="p">,</span> <span class="n">value</span><span class="p">)</span> |
| <span class="k">return</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Option</span><span class="p">(</span><span class="n">name</span><span class="o">=</span><span class="n">name</span><span class="p">,</span> <span class="nb">type</span><span class="o">=</span><span class="n">type_proto</span><span class="p">,</span> <span class="n">value</span><span class="o">=</span><span class="n">value_proto</span><span class="p">)</span></div> |
| |
| <div class="viewcode-block" id="SchemaTranslation.typing_from_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.typing_from_runner_api">[docs]</a> <span class="k">def</span> <span class="nf">typing_from_runner_api</span><span class="p">(</span> |
| <span class="bp">self</span><span class="p">,</span> <span class="n">fieldtype_proto</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">)</span> <span class="o">-></span> <span class="nb">type</span><span class="p">:</span> |
| <span class="k">if</span> <span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">nullable</span><span class="p">:</span> |
| <span class="c1"># In order to determine the inner type, create a copy of fieldtype_proto</span> |
| <span class="c1"># with nullable=False and pass back to typing_from_runner_api</span> |
| <span class="n">base_type</span> <span class="o">=</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">FieldType</span><span class="p">()</span> |
| <span class="n">base_type</span><span class="o">.</span><span class="n">CopyFrom</span><span class="p">(</span><span class="n">fieldtype_proto</span><span class="p">)</span> |
| <span class="n">base_type</span><span class="o">.</span><span class="n">nullable</span> <span class="o">=</span> <span class="kc">False</span> |
| <span class="n">base</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">typing_from_runner_api</span><span class="p">(</span><span class="n">base_type</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">base</span> <span class="o">==</span> <span class="n">Any</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">base</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">Optional</span><span class="p">[</span><span class="n">base</span><span class="p">]</span> |
| |
| <span class="n">type_info</span> <span class="o">=</span> <span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">WhichOneof</span><span class="p">(</span><span class="s2">"type_info"</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">type_info</span> <span class="o">==</span> <span class="s2">"atomic_type"</span><span class="p">:</span> |
| <span class="k">try</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">ATOMIC_TYPE_TO_PRIMITIVE</span><span class="p">[</span><span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">atomic_type</span><span class="p">]</span> |
| <span class="k">except</span> <span class="ne">KeyError</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"Unsupported atomic type: </span><span class="si">{0}</span><span class="s2">"</span><span class="o">.</span><span class="n">format</span><span class="p">(</span><span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">atomic_type</span><span class="p">))</span> |
| <span class="k">elif</span> <span class="n">type_info</span> <span class="o">==</span> <span class="s2">"array_type"</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">Sequence</span><span class="p">[</span><span class="bp">self</span><span class="o">.</span><span class="n">typing_from_runner_api</span><span class="p">(</span> |
| <span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">array_type</span><span class="o">.</span><span class="n">element_type</span><span class="p">)]</span> |
| <span class="k">elif</span> <span class="n">type_info</span> <span class="o">==</span> <span class="s2">"map_type"</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">Mapping</span><span class="p">[</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">typing_from_runner_api</span><span class="p">(</span><span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">map_type</span><span class="o">.</span><span class="n">key_type</span><span class="p">),</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">typing_from_runner_api</span><span class="p">(</span><span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">map_type</span><span class="o">.</span><span class="n">value_type</span><span class="p">)]</span> |
| <span class="k">elif</span> <span class="n">type_info</span> <span class="o">==</span> <span class="s2">"row_type"</span><span class="p">:</span> |
| <span class="n">schema</span> <span class="o">=</span> <span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">row_type</span><span class="o">.</span><span class="n">schema</span> |
| <span class="n">schema_options</span> <span class="o">=</span> <span class="p">[</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">option_from_runner_api</span><span class="p">(</span><span class="n">option_proto</span><span class="p">)</span> |
| <span class="k">for</span> <span class="n">option_proto</span> <span class="ow">in</span> <span class="n">schema</span><span class="o">.</span><span class="n">options</span> |
| <span class="p">]</span> |
| <span class="n">field_options</span> <span class="o">=</span> <span class="p">{</span> |
| <span class="n">field</span><span class="o">.</span><span class="n">name</span><span class="p">:</span> <span class="p">[</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">option_from_runner_api</span><span class="p">(</span><span class="n">option_proto</span><span class="p">)</span> |
| <span class="k">for</span> <span class="n">option_proto</span> <span class="ow">in</span> <span class="n">field</span><span class="o">.</span><span class="n">options</span> |
| <span class="p">]</span> |
| <span class="k">for</span> <span class="n">field</span> <span class="ow">in</span> <span class="n">schema</span><span class="o">.</span><span class="n">fields</span> <span class="k">if</span> <span class="n">field</span><span class="o">.</span><span class="n">options</span> |
| <span class="p">}</span> |
| <span class="c1"># First look for user type in the registry</span> |
| <span class="n">user_type</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">schema_registry</span><span class="o">.</span><span class="n">get_typing_by_id</span><span class="p">(</span><span class="n">schema</span><span class="o">.</span><span class="n">id</span><span class="p">)</span> |
| |
| <span class="k">if</span> <span class="n">user_type</span> <span class="ow">is</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="c1"># If not in SDK options (the coder likely came from another SDK),</span> |
| <span class="c1"># generate a NamedTuple type to use.</span> |
| |
| <span class="n">fields</span> <span class="o">=</span> <span class="n">named_fields_from_schema</span><span class="p">(</span><span class="n">schema</span><span class="p">)</span> |
| <span class="n">result</span> <span class="o">=</span> <span class="n">row_type</span><span class="o">.</span><span class="n">RowTypeConstraint</span><span class="o">.</span><span class="n">from_fields</span><span class="p">(</span> |
| <span class="n">fields</span><span class="o">=</span><span class="n">fields</span><span class="p">,</span> |
| <span class="n">schema_id</span><span class="o">=</span><span class="n">schema</span><span class="o">.</span><span class="n">id</span><span class="p">,</span> |
| <span class="n">schema_options</span><span class="o">=</span><span class="n">schema_options</span><span class="p">,</span> |
| <span class="n">field_options</span><span class="o">=</span><span class="n">field_options</span><span class="p">,</span> |
| <span class="n">schema_registry</span><span class="o">=</span><span class="bp">self</span><span class="o">.</span><span class="n">schema_registry</span><span class="p">,</span> |
| <span class="p">)</span> |
| <span class="k">return</span> <span class="n">result</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">row_type</span><span class="o">.</span><span class="n">RowTypeConstraint</span><span class="o">.</span><span class="n">from_user_type</span><span class="p">(</span> |
| <span class="n">user_type</span><span class="p">,</span> |
| <span class="n">schema_options</span><span class="o">=</span><span class="n">schema_options</span><span class="p">,</span> |
| <span class="n">field_options</span><span class="o">=</span><span class="n">field_options</span><span class="p">)</span> |
| |
| <span class="k">elif</span> <span class="n">type_info</span> <span class="o">==</span> <span class="s2">"logical_type"</span><span class="p">:</span> |
| <span class="k">if</span> <span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">logical_type</span><span class="o">.</span><span class="n">urn</span> <span class="o">==</span> <span class="n">PYTHON_ANY_URN</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">Any</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">LogicalType</span><span class="o">.</span><span class="n">from_runner_api</span><span class="p">(</span> |
| <span class="n">fieldtype_proto</span><span class="o">.</span><span class="n">logical_type</span><span class="p">)</span><span class="o">.</span><span class="n">language_type</span><span class="p">()</span> |
| |
| <span class="k">else</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span><span class="sa">f</span><span class="s2">"Unrecognized type_info: </span><span class="si">{</span><span class="n">type_info</span><span class="si">!r}</span><span class="s2">"</span><span class="p">)</span></div> |
| |
| <div class="viewcode-block" id="SchemaTranslation.named_tuple_from_schema"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.SchemaTranslation.named_tuple_from_schema">[docs]</a> <span class="k">def</span> <span class="nf">named_tuple_from_schema</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">schema</span><span class="p">:</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Schema</span><span class="p">)</span> <span class="o">-></span> <span class="nb">type</span><span class="p">:</span> |
| <span class="kn">from</span> <span class="nn">apache_beam</span> <span class="kn">import</span> <span class="n">coders</span> |
| |
| <span class="n">type_name</span> <span class="o">=</span> <span class="s1">'BeamSchema_</span><span class="si">{}</span><span class="s1">'</span><span class="o">.</span><span class="n">format</span><span class="p">(</span><span class="n">schema</span><span class="o">.</span><span class="n">id</span><span class="o">.</span><span class="n">replace</span><span class="p">(</span><span class="s1">'-'</span><span class="p">,</span> <span class="s1">'_'</span><span class="p">))</span> |
| |
| <span class="n">subfields</span> <span class="o">=</span> <span class="p">[]</span> |
| <span class="k">for</span> <span class="n">field</span> <span class="ow">in</span> <span class="n">schema</span><span class="o">.</span><span class="n">fields</span><span class="p">:</span> |
| <span class="k">try</span><span class="p">:</span> |
| <span class="n">field_py_type</span> <span class="o">=</span> <span class="bp">self</span><span class="o">.</span><span class="n">typing_from_runner_api</span><span class="p">(</span><span class="n">field</span><span class="o">.</span><span class="n">type</span><span class="p">)</span> |
| <span class="k">if</span> <span class="nb">isinstance</span><span class="p">(</span><span class="n">field_py_type</span><span class="p">,</span> <span class="n">row_type</span><span class="o">.</span><span class="n">RowTypeConstraint</span><span class="p">):</span> |
| <span class="n">field_py_type</span> <span class="o">=</span> <span class="n">field_py_type</span><span class="o">.</span><span class="n">user_type</span> |
| <span class="k">except</span> <span class="ne">ValueError</span> <span class="k">as</span> <span class="n">e</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"Failed to decode schema due to an issue with Field proto:</span><span class="se">\n\n</span><span class="s2">"</span> |
| <span class="sa">f</span><span class="s2">"</span><span class="si">{</span><span class="n">text_format</span><span class="o">.</span><span class="n">MessageToString</span><span class="p">(</span><span class="n">field</span><span class="p">)</span><span class="si">}</span><span class="s2">"</span><span class="p">)</span> <span class="kn">from</span> <span class="nn">e</span> |
| |
| <span class="n">subfields</span><span class="o">.</span><span class="n">append</span><span class="p">((</span><span class="n">field</span><span class="o">.</span><span class="n">name</span><span class="p">,</span> <span class="n">field_py_type</span><span class="p">))</span> |
| |
| <span class="n">user_type</span> <span class="o">=</span> <span class="n">NamedTuple</span><span class="p">(</span><span class="n">type_name</span><span class="p">,</span> <span class="n">subfields</span><span class="p">)</span> |
| |
| <span class="c1"># Define a reduce function, otherwise these types can't be pickled</span> |
| <span class="c1"># (See BEAM-9574)</span> |
| <span class="nb">setattr</span><span class="p">(</span> |
| <span class="n">user_type</span><span class="p">,</span> |
| <span class="s1">'__reduce__'</span><span class="p">,</span> |
| <span class="n">_named_tuple_reduce_method</span><span class="p">(</span><span class="n">schema</span><span class="o">.</span><span class="n">SerializeToString</span><span class="p">()))</span> |
| <span class="nb">setattr</span><span class="p">(</span><span class="n">user_type</span><span class="p">,</span> <span class="n">row_type</span><span class="o">.</span><span class="n">_BEAM_SCHEMA_ID</span><span class="p">,</span> <span class="n">schema</span><span class="o">.</span><span class="n">id</span><span class="p">)</span> |
| |
| <span class="bp">self</span><span class="o">.</span><span class="n">schema_registry</span><span class="o">.</span><span class="n">add</span><span class="p">(</span><span class="n">user_type</span><span class="p">,</span> <span class="n">schema</span><span class="p">)</span> |
| <span class="n">coders</span><span class="o">.</span><span class="n">registry</span><span class="o">.</span><span class="n">register_coder</span><span class="p">(</span><span class="n">user_type</span><span class="p">,</span> <span class="n">coders</span><span class="o">.</span><span class="n">RowCoder</span><span class="p">)</span> |
| |
| <span class="k">return</span> <span class="n">user_type</span></div></div> |
| |
| |
| <span class="k">def</span> <span class="nf">_named_tuple_reduce_method</span><span class="p">(</span><span class="n">serialized_schema</span><span class="p">):</span> |
| <span class="k">def</span> <span class="nf">__reduce__</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">_hydrate_namedtuple_instance</span><span class="p">,</span> <span class="p">(</span><span class="n">serialized_schema</span><span class="p">,</span> <span class="nb">tuple</span><span class="p">(</span><span class="bp">self</span><span class="p">))</span> |
| |
| <span class="k">return</span> <span class="n">__reduce__</span> |
| |
| |
| <span class="k">def</span> <span class="nf">_hydrate_namedtuple_instance</span><span class="p">(</span><span class="n">encoded_schema</span><span class="p">,</span> <span class="n">values</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">named_tuple_from_schema</span><span class="p">(</span> |
| <span class="n">proto_utils</span><span class="o">.</span><span class="n">parse_Bytes</span><span class="p">(</span><span class="n">encoded_schema</span><span class="p">,</span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Schema</span><span class="p">))(</span><span class="o">*</span><span class="n">values</span><span class="p">)</span> |
| |
| |
| <div class="viewcode-block" id="named_tuple_from_schema"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.named_tuple_from_schema">[docs]</a><span class="k">def</span> <span class="nf">named_tuple_from_schema</span><span class="p">(</span> |
| <span class="n">schema</span><span class="p">,</span> <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span><span class="p">)</span> <span class="o">-></span> <span class="nb">type</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">SchemaTranslation</span><span class="p">(</span> |
| <span class="n">schema_registry</span><span class="o">=</span><span class="n">schema_registry</span><span class="p">)</span><span class="o">.</span><span class="n">named_tuple_from_schema</span><span class="p">(</span><span class="n">schema</span><span class="p">)</span></div> |
| |
| |
| <div class="viewcode-block" id="named_tuple_to_schema"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.named_tuple_to_schema">[docs]</a><span class="k">def</span> <span class="nf">named_tuple_to_schema</span><span class="p">(</span> |
| <span class="n">named_tuple</span><span class="p">,</span> |
| <span class="n">schema_registry</span><span class="p">:</span> <span class="n">SchemaTypeRegistry</span> <span class="o">=</span> <span class="n">SCHEMA_REGISTRY</span><span class="p">)</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Schema</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">typing_to_runner_api</span><span class="p">(</span><span class="n">named_tuple</span><span class="p">,</span> <span class="n">schema_registry</span><span class="p">)</span><span class="o">.</span><span class="n">row_type</span><span class="o">.</span><span class="n">schema</span></div> |
| |
| |
| <div class="viewcode-block" id="schema_from_element_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.schema_from_element_type">[docs]</a><span class="k">def</span> <span class="nf">schema_from_element_type</span><span class="p">(</span><span class="n">element_type</span><span class="p">:</span> <span class="nb">type</span><span class="p">)</span> <span class="o">-></span> <span class="n">schema_pb2</span><span class="o">.</span><span class="n">Schema</span><span class="p">:</span> |
| <span class="w"> </span><span class="sd">"""Get a schema for the given PCollection element_type.</span> |
| |
| <span class="sd"> Returns schema as a list of (name, python_type) tuples"""</span> |
| <span class="k">if</span> <span class="nb">isinstance</span><span class="p">(</span><span class="n">element_type</span><span class="p">,</span> <span class="n">row_type</span><span class="o">.</span><span class="n">RowTypeConstraint</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">named_fields_to_schema</span><span class="p">(</span><span class="n">element_type</span><span class="o">.</span><span class="n">_fields</span><span class="p">)</span> |
| <span class="k">elif</span> <span class="n">match_is_named_tuple</span><span class="p">(</span><span class="n">element_type</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">named_tuple_to_schema</span><span class="p">(</span><span class="n">element_type</span><span class="p">)</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">TypeError</span><span class="p">(</span> |
| <span class="sa">f</span><span class="s2">"Could not determine schema for type hint </span><span class="si">{</span><span class="n">element_type</span><span class="si">!r}</span><span class="s2">. Did you "</span> |
| <span class="s2">"mean to create a schema-aware PCollection? See "</span> |
| <span class="s2">"https://s.apache.org/beam-python-schemas"</span><span class="p">)</span></div> |
| |
| |
| <div class="viewcode-block" id="named_fields_from_element_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.named_fields_from_element_type">[docs]</a><span class="k">def</span> <span class="nf">named_fields_from_element_type</span><span class="p">(</span> |
| <span class="n">element_type</span><span class="p">:</span> <span class="nb">type</span><span class="p">)</span> <span class="o">-></span> <span class="n">List</span><span class="p">[</span><span class="n">Tuple</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="nb">type</span><span class="p">]]:</span> |
| <span class="k">return</span> <span class="n">named_fields_from_schema</span><span class="p">(</span><span class="n">schema_from_element_type</span><span class="p">(</span><span class="n">element_type</span><span class="p">))</span></div> |
| |
| |
| <span class="c1"># Registry of typings for a schema by UUID</span> |
| <div class="viewcode-block" id="LogicalTypeRegistry"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalTypeRegistry">[docs]</a><span class="k">class</span> <span class="nc">LogicalTypeRegistry</span><span class="p">(</span><span class="nb">object</span><span class="p">):</span> |
| <span class="k">def</span> <span class="fm">__init__</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">by_urn</span> <span class="o">=</span> <span class="p">{}</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">by_logical_type</span> <span class="o">=</span> <span class="p">{}</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">by_language_type</span> <span class="o">=</span> <span class="p">{}</span> |
| |
| <div class="viewcode-block" id="LogicalTypeRegistry.add"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalTypeRegistry.add">[docs]</a> <span class="k">def</span> <span class="nf">add</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">urn</span><span class="p">,</span> <span class="n">logical_type</span><span class="p">):</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">by_urn</span><span class="p">[</span><span class="n">urn</span><span class="p">]</span> <span class="o">=</span> <span class="n">logical_type</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">by_logical_type</span><span class="p">[</span><span class="n">logical_type</span><span class="p">]</span> <span class="o">=</span> <span class="n">urn</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">by_language_type</span><span class="p">[</span><span class="n">logical_type</span><span class="o">.</span><span class="n">language_type</span><span class="p">()]</span> <span class="o">=</span> <span class="n">logical_type</span></div> |
| |
| <div class="viewcode-block" id="LogicalTypeRegistry.get_logical_type_by_urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalTypeRegistry.get_logical_type_by_urn">[docs]</a> <span class="k">def</span> <span class="nf">get_logical_type_by_urn</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">urn</span><span class="p">):</span> |
| <span class="k">return</span> <span class="bp">self</span><span class="o">.</span><span class="n">by_urn</span><span class="o">.</span><span class="n">get</span><span class="p">(</span><span class="n">urn</span><span class="p">,</span> <span class="kc">None</span><span class="p">)</span></div> |
| |
| <div class="viewcode-block" id="LogicalTypeRegistry.get_urn_by_logial_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalTypeRegistry.get_urn_by_logial_type">[docs]</a> <span class="k">def</span> <span class="nf">get_urn_by_logial_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">logical_type</span><span class="p">):</span> |
| <span class="k">return</span> <span class="bp">self</span><span class="o">.</span><span class="n">by_logical_type</span><span class="o">.</span><span class="n">get</span><span class="p">(</span><span class="n">logical_type</span><span class="p">,</span> <span class="kc">None</span><span class="p">)</span></div> |
| |
| <div class="viewcode-block" id="LogicalTypeRegistry.get_logical_type_by_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalTypeRegistry.get_logical_type_by_language_type">[docs]</a> <span class="k">def</span> <span class="nf">get_logical_type_by_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">representation_type</span><span class="p">):</span> |
| <span class="k">return</span> <span class="bp">self</span><span class="o">.</span><span class="n">by_language_type</span><span class="o">.</span><span class="n">get</span><span class="p">(</span><span class="n">representation_type</span><span class="p">,</span> <span class="kc">None</span><span class="p">)</span></div></div> |
| |
| |
| <span class="n">LanguageT</span> <span class="o">=</span> <span class="n">TypeVar</span><span class="p">(</span><span class="s1">'LanguageT'</span><span class="p">)</span> |
| <span class="n">RepresentationT</span> <span class="o">=</span> <span class="n">TypeVar</span><span class="p">(</span><span class="s1">'RepresentationT'</span><span class="p">)</span> |
| <span class="n">ArgT</span> <span class="o">=</span> <span class="n">TypeVar</span><span class="p">(</span><span class="s1">'ArgT'</span><span class="p">)</span> |
| |
| |
| <div class="viewcode-block" id="LogicalType"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType">[docs]</a><span class="k">class</span> <span class="nc">LogicalType</span><span class="p">(</span><span class="n">Generic</span><span class="p">[</span><span class="n">LanguageT</span><span class="p">,</span> <span class="n">RepresentationT</span><span class="p">,</span> <span class="n">ArgT</span><span class="p">]):</span> |
| <span class="n">_known_logical_types</span> <span class="o">=</span> <span class="n">LogicalTypeRegistry</span><span class="p">()</span> |
| |
| <div class="viewcode-block" id="LogicalType.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> str</span> |
| |
| <span class="w"> </span><span class="sd">"""Return the URN used to identify this logical type"""</span> |
| <span class="k">raise</span> <span class="ne">NotImplementedError</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="LogicalType.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| |
| <span class="w"> </span><span class="sd">"""Return the language type this LogicalType encodes.</span> |
| |
| <span class="sd"> The returned type should match LanguageT"""</span> |
| <span class="k">raise</span> <span class="ne">NotImplementedError</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="LogicalType.representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.representation_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">representation_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| |
| <span class="w"> </span><span class="sd">"""Return the type of the representation this LogicalType uses to encode the</span> |
| <span class="sd"> language type.</span> |
| |
| <span class="sd"> The returned type should match RepresentationT"""</span> |
| <span class="k">raise</span> <span class="ne">NotImplementedError</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="LogicalType.argument_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.argument_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">argument_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| |
| <span class="w"> </span><span class="sd">"""Return the type of the argument used for variations of this LogicalType.</span> |
| |
| <span class="sd"> The returned type should match ArgT"""</span> |
| <span class="k">raise</span> <span class="ne">NotImplementedError</span><span class="p">(</span><span class="bp">cls</span><span class="p">)</span></div> |
| |
| <div class="viewcode-block" id="LogicalType.argument"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.argument">[docs]</a> <span class="k">def</span> <span class="nf">argument</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="c1"># type: () -> ArgT</span> |
| |
| <span class="w"> </span><span class="sd">"""Return the argument for this instance of the LogicalType."""</span> |
| <span class="k">raise</span> <span class="ne">NotImplementedError</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="LogicalType.to_representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.to_representation_type">[docs]</a> <span class="k">def</span> <span class="nf">to_representation_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (LanguageT) -> RepresentationT</span> |
| |
| <span class="w"> </span><span class="sd">"""Convert an instance of LanguageT to RepresentationT."""</span> |
| <span class="k">raise</span> <span class="ne">NotImplementedError</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="LogicalType.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (RepresentationT) -> LanguageT</span> |
| |
| <span class="w"> </span><span class="sd">"""Convert an instance of RepresentationT to LanguageT."""</span> |
| <span class="k">raise</span> <span class="ne">NotImplementedError</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="LogicalType.register_logical_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.register_logical_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">register_logical_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">,</span> <span class="n">logical_type_cls</span><span class="p">):</span> |
| <span class="w"> </span><span class="sd">"""Register an implementation of LogicalType."""</span> |
| <span class="bp">cls</span><span class="o">.</span><span class="n">_known_logical_types</span><span class="o">.</span><span class="n">add</span><span class="p">(</span><span class="n">logical_type_cls</span><span class="o">.</span><span class="n">urn</span><span class="p">(),</span> <span class="n">logical_type_cls</span><span class="p">)</span> |
| <span class="k">return</span> <span class="n">logical_type_cls</span></div> |
| |
| <div class="viewcode-block" id="LogicalType.from_typing"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.from_typing">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">from_typing</span><span class="p">(</span><span class="bp">cls</span><span class="p">,</span> <span class="n">typ</span><span class="p">):</span> |
| <span class="c1"># type: (type) -> LogicalType</span> |
| |
| <span class="w"> </span><span class="sd">"""Construct an instance of a registered LogicalType implementation given a</span> |
| <span class="sd"> typing.</span> |
| |
| <span class="sd"> Raises ValueError if no registered LogicalType implementation can encode the</span> |
| <span class="sd"> given typing."""</span> |
| |
| <span class="n">logical_type</span> <span class="o">=</span> <span class="bp">cls</span><span class="o">.</span><span class="n">_known_logical_types</span><span class="o">.</span><span class="n">get_logical_type_by_language_type</span><span class="p">(</span> |
| <span class="n">typ</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">logical_type</span> <span class="ow">is</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span><span class="s2">"No logical type registered for typing '</span><span class="si">%s</span><span class="s2">'"</span> <span class="o">%</span> <span class="n">typ</span><span class="p">)</span> |
| |
| <span class="k">return</span> <span class="n">logical_type</span><span class="o">.</span><span class="n">_from_typing</span><span class="p">(</span><span class="n">typ</span><span class="p">)</span></div> |
| |
| <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">_from_typing</span><span class="p">(</span><span class="bp">cls</span><span class="p">,</span> <span class="n">typ</span><span class="p">):</span> |
| <span class="c1"># type: (type) -> LogicalType</span> |
| |
| <span class="w"> </span><span class="sd">"""Construct an instance of this LogicalType implementation given a typing.</span> |
| <span class="sd"> """</span> |
| <span class="k">raise</span> <span class="ne">NotImplementedError</span><span class="p">()</span> |
| |
| <div class="viewcode-block" id="LogicalType.from_runner_api"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.LogicalType.from_runner_api">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">from_runner_api</span><span class="p">(</span><span class="bp">cls</span><span class="p">,</span> <span class="n">logical_type_proto</span><span class="p">):</span> |
| <span class="c1"># type: (schema_pb2.LogicalType) -> LogicalType</span> |
| |
| <span class="w"> </span><span class="sd">"""Construct an instance of a registered LogicalType implementation given a</span> |
| <span class="sd"> proto LogicalType.</span> |
| |
| <span class="sd"> Raises ValueError if no LogicalType registered for the given URN.</span> |
| <span class="sd"> """</span> |
| <span class="n">logical_type</span> <span class="o">=</span> <span class="bp">cls</span><span class="o">.</span><span class="n">_known_logical_types</span><span class="o">.</span><span class="n">get_logical_type_by_urn</span><span class="p">(</span> |
| <span class="n">logical_type_proto</span><span class="o">.</span><span class="n">urn</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">logical_type</span> <span class="ow">is</span> <span class="kc">None</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"No logical type registered for URN '</span><span class="si">%s</span><span class="s2">'"</span> <span class="o">%</span> <span class="n">logical_type_proto</span><span class="o">.</span><span class="n">urn</span><span class="p">)</span> |
| <span class="k">if</span> <span class="ow">not</span> <span class="n">logical_type_proto</span><span class="o">.</span><span class="n">HasField</span><span class="p">(</span> |
| <span class="s2">"argument_type"</span><span class="p">)</span> <span class="ow">or</span> <span class="ow">not</span> <span class="n">logical_type_proto</span><span class="o">.</span><span class="n">HasField</span><span class="p">(</span><span class="s2">"argument"</span><span class="p">):</span> |
| <span class="c1"># logical type_proto without argument</span> |
| <span class="k">return</span> <span class="n">logical_type</span><span class="p">()</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="k">try</span><span class="p">:</span> |
| <span class="n">argument</span> <span class="o">=</span> <span class="n">value_from_runner_api</span><span class="p">(</span> |
| <span class="n">logical_type_proto</span><span class="o">.</span><span class="n">argument_type</span><span class="p">,</span> <span class="n">logical_type_proto</span><span class="o">.</span><span class="n">argument</span><span class="p">)</span> |
| <span class="k">except</span> <span class="ne">ValueError</span><span class="p">:</span> |
| <span class="c1"># TODO(https://github.com/apache/beam/issues/23373): Complete support</span> |
| <span class="c1"># for logical types that require arguments beyond atomic type.</span> |
| <span class="c1"># For now, skip arguments.</span> |
| <span class="n">_LOGGER</span><span class="o">.</span><span class="n">warning</span><span class="p">(</span> |
| <span class="s1">'Logical type </span><span class="si">%s</span><span class="s1"> with argument is currently unsupported. '</span> |
| <span class="s1">'Argument values are omitted'</span><span class="p">,</span> |
| <span class="n">logical_type_proto</span><span class="o">.</span><span class="n">urn</span><span class="p">)</span> |
| <span class="k">return</span> <span class="n">logical_type</span><span class="p">()</span> |
| <span class="k">return</span> <span class="n">logical_type</span><span class="p">(</span><span class="n">argument</span><span class="p">)</span></div></div> |
| |
| |
| <div class="viewcode-block" id="NoArgumentLogicalType"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.NoArgumentLogicalType">[docs]</a><span class="k">class</span> <span class="nc">NoArgumentLogicalType</span><span class="p">(</span><span class="n">LogicalType</span><span class="p">[</span><span class="n">LanguageT</span><span class="p">,</span> <span class="n">RepresentationT</span><span class="p">,</span> <span class="kc">None</span><span class="p">]):</span> |
| <div class="viewcode-block" id="NoArgumentLogicalType.argument_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.NoArgumentLogicalType.argument_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">argument_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| <span class="k">return</span> <span class="kc">None</span></div> |
| |
| <div class="viewcode-block" id="NoArgumentLogicalType.argument"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.NoArgumentLogicalType.argument">[docs]</a> <span class="k">def</span> <span class="nf">argument</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="c1"># type: () -> ArgT</span> |
| <span class="k">return</span> <span class="kc">None</span></div> |
| |
| <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">_from_typing</span><span class="p">(</span><span class="bp">cls</span><span class="p">,</span> <span class="n">typ</span><span class="p">):</span> |
| <span class="c1"># type: (type) -> LogicalType</span> |
| |
| <span class="c1"># Since there's no argument, there can be no additional information encoded</span> |
| <span class="c1"># in the typing. Just construct an instance.</span> |
| <span class="k">return</span> <span class="bp">cls</span><span class="p">()</span></div> |
| |
| |
| <div class="viewcode-block" id="PassThroughLogicalType"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PassThroughLogicalType">[docs]</a><span class="k">class</span> <span class="nc">PassThroughLogicalType</span><span class="p">(</span><span class="n">LogicalType</span><span class="p">[</span><span class="n">LanguageT</span><span class="p">,</span> <span class="n">LanguageT</span><span class="p">,</span> <span class="n">ArgT</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""A base class for LogicalTypes that use the same type as the underlying</span> |
| <span class="sd"> representation type.</span> |
| <span class="sd"> """</span> |
| <div class="viewcode-block" id="PassThroughLogicalType.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PassThroughLogicalType.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">value</span></div> |
| |
| <div class="viewcode-block" id="PassThroughLogicalType.representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PassThroughLogicalType.representation_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">representation_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| <span class="k">return</span> <span class="bp">cls</span><span class="o">.</span><span class="n">language_type</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="PassThroughLogicalType.to_representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PassThroughLogicalType.to_representation_type">[docs]</a> <span class="k">def</span> <span class="nf">to_representation_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">value</span></div> |
| |
| <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">_from_typing</span><span class="p">(</span><span class="bp">cls</span><span class="p">,</span> <span class="n">typ</span><span class="p">):</span> |
| <span class="c1"># type: (type) -> LogicalType</span> |
| <span class="c1"># TODO(https://github.com/apache/beam/issues/23373): enable argument</span> |
| <span class="k">return</span> <span class="bp">cls</span><span class="p">()</span></div> |
| |
| |
| <span class="n">MicrosInstantRepresentation</span> <span class="o">=</span> <span class="n">NamedTuple</span><span class="p">(</span> |
| <span class="s1">'MicrosInstantRepresentation'</span><span class="p">,</span> <span class="p">[(</span><span class="s1">'seconds'</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int64</span><span class="p">),</span> |
| <span class="p">(</span><span class="s1">'micros'</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int64</span><span class="p">)])</span> |
| |
| |
| <div class="viewcode-block" id="MillisInstant"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MillisInstant">[docs]</a><span class="nd">@LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span> |
| <span class="k">class</span> <span class="nc">MillisInstant</span><span class="p">(</span><span class="n">NoArgumentLogicalType</span><span class="p">[</span><span class="n">Timestamp</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int64</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""Millisecond-precision instant logical type handles values consistent with</span> |
| <span class="sd"> that encoded by ``InstantCoder`` in the Java SDK.</span> |
| |
| <span class="sd"> This class handles :class:`apache_beam.utils.timestamp.Timestamp` language</span> |
| <span class="sd"> type as :class:`MicrosInstant`, but it only provides millisecond precision,</span> |
| <span class="sd"> because it is aimed to handle data encoded by Java sdk's InstantCoder which</span> |
| <span class="sd"> has same precision level.</span> |
| |
| <span class="sd"> Timestamp is handled by `MicrosInstant` by default. In some scenario, such as</span> |
| <span class="sd"> read from cross-language transform with rows containing InstantCoder encoded</span> |
| <span class="sd"> timestamps, one may need to override the mapping of Timetamp to MillisInstant.</span> |
| <span class="sd"> To do this, re-register this class with</span> |
| <span class="sd"> :func:`~LogicalType.register_logical_type`.</span> |
| <span class="sd"> """</span> |
| <div class="viewcode-block" id="MillisInstant.representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MillisInstant.representation_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">representation_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| <span class="k">return</span> <span class="n">np</span><span class="o">.</span><span class="n">int64</span></div> |
| |
| <div class="viewcode-block" id="MillisInstant.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MillisInstant.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">common_urns</span><span class="o">.</span><span class="n">millis_instant</span><span class="o">.</span><span class="n">urn</span></div> |
| |
| <div class="viewcode-block" id="MillisInstant.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MillisInstant.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">Timestamp</span></div> |
| |
| <div class="viewcode-block" id="MillisInstant.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MillisInstant.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (np.int64) -> Timestamp</span> |
| |
| <span class="c1"># value shifted as in apache_beams.coders.coder_impl.TimestampCoderImpl</span> |
| <span class="k">if</span> <span class="n">value</span> <span class="o"><</span> <span class="mi">0</span><span class="p">:</span> |
| <span class="n">millis</span> <span class="o">=</span> <span class="nb">int</span><span class="p">(</span><span class="n">value</span><span class="p">)</span> <span class="o">+</span> <span class="p">(</span><span class="mi">1</span> <span class="o"><<</span> <span class="mi">63</span><span class="p">)</span> |
| <span class="k">else</span><span class="p">:</span> |
| <span class="n">millis</span> <span class="o">=</span> <span class="nb">int</span><span class="p">(</span><span class="n">value</span><span class="p">)</span> <span class="o">-</span> <span class="p">(</span><span class="mi">1</span> <span class="o"><<</span> <span class="mi">63</span><span class="p">)</span> |
| |
| <span class="k">return</span> <span class="n">Timestamp</span><span class="p">(</span><span class="n">micros</span><span class="o">=</span><span class="n">millis</span> <span class="o">*</span> <span class="mi">1000</span><span class="p">)</span></div></div> |
| |
| |
| <span class="c1"># Make sure MicrosInstant is registered after MillisInstant so that it</span> |
| <span class="c1"># overwrites the mapping of Timestamp language type representation choice and</span> |
| <span class="c1"># thus does not lose microsecond precision inside python sdk.</span> |
| <div class="viewcode-block" id="MicrosInstant"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MicrosInstant">[docs]</a><span class="nd">@LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span> |
| <span class="k">class</span> <span class="nc">MicrosInstant</span><span class="p">(</span><span class="n">NoArgumentLogicalType</span><span class="p">[</span><span class="n">Timestamp</span><span class="p">,</span> |
| <span class="n">MicrosInstantRepresentation</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""Microsecond-precision instant logical type that handles ``Timestamp``."""</span> |
| <div class="viewcode-block" id="MicrosInstant.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MicrosInstant.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">common_urns</span><span class="o">.</span><span class="n">micros_instant</span><span class="o">.</span><span class="n">urn</span></div> |
| |
| <div class="viewcode-block" id="MicrosInstant.representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MicrosInstant.representation_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">representation_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| <span class="k">return</span> <span class="n">MicrosInstantRepresentation</span></div> |
| |
| <div class="viewcode-block" id="MicrosInstant.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MicrosInstant.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">Timestamp</span></div> |
| |
| <div class="viewcode-block" id="MicrosInstant.to_representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MicrosInstant.to_representation_type">[docs]</a> <span class="k">def</span> <span class="nf">to_representation_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (Timestamp) -> MicrosInstantRepresentation</span> |
| <span class="k">return</span> <span class="n">MicrosInstantRepresentation</span><span class="p">(</span> |
| <span class="n">value</span><span class="o">.</span><span class="n">micros</span> <span class="o">//</span> <span class="mi">1000000</span><span class="p">,</span> <span class="n">value</span><span class="o">.</span><span class="n">micros</span> <span class="o">%</span> <span class="mi">1000000</span><span class="p">)</span></div> |
| |
| <div class="viewcode-block" id="MicrosInstant.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.MicrosInstant.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (MicrosInstantRepresentation) -> Timestamp</span> |
| <span class="k">return</span> <span class="n">Timestamp</span><span class="p">(</span><span class="n">seconds</span><span class="o">=</span><span class="nb">int</span><span class="p">(</span><span class="n">value</span><span class="o">.</span><span class="n">seconds</span><span class="p">),</span> <span class="n">micros</span><span class="o">=</span><span class="nb">int</span><span class="p">(</span><span class="n">value</span><span class="o">.</span><span class="n">micros</span><span class="p">))</span></div></div> |
| |
| |
| <div class="viewcode-block" id="PythonCallable"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PythonCallable">[docs]</a><span class="nd">@LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span> |
| <span class="k">class</span> <span class="nc">PythonCallable</span><span class="p">(</span><span class="n">NoArgumentLogicalType</span><span class="p">[</span><span class="n">PythonCallableWithSource</span><span class="p">,</span> <span class="nb">str</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""A logical type for PythonCallableSource objects."""</span> |
| <div class="viewcode-block" id="PythonCallable.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PythonCallable.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">common_urns</span><span class="o">.</span><span class="n">python_callable</span><span class="o">.</span><span class="n">urn</span></div> |
| |
| <div class="viewcode-block" id="PythonCallable.representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PythonCallable.representation_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">representation_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| <span class="k">return</span> <span class="nb">str</span></div> |
| |
| <div class="viewcode-block" id="PythonCallable.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PythonCallable.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">PythonCallableWithSource</span></div> |
| |
| <div class="viewcode-block" id="PythonCallable.to_representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PythonCallable.to_representation_type">[docs]</a> <span class="k">def</span> <span class="nf">to_representation_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (PythonCallableWithSource) -> str</span> |
| <span class="k">return</span> <span class="n">value</span><span class="o">.</span><span class="n">get_source</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="PythonCallable.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.PythonCallable.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (str) -> PythonCallableWithSource</span> |
| <span class="k">return</span> <span class="n">PythonCallableWithSource</span><span class="p">(</span><span class="n">value</span><span class="p">)</span></div></div> |
| |
| |
| <span class="n">FixedPrecisionDecimalArgumentRepresentation</span> <span class="o">=</span> <span class="n">NamedTuple</span><span class="p">(</span> |
| <span class="s1">'FixedPrecisionDecimalArgumentRepresentation'</span><span class="p">,</span> <span class="p">[(</span><span class="s1">'precision'</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">),</span> |
| <span class="p">(</span><span class="s1">'scale'</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">)])</span> |
| |
| |
| <div class="viewcode-block" id="DecimalLogicalType"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.DecimalLogicalType">[docs]</a><span class="k">class</span> <span class="nc">DecimalLogicalType</span><span class="p">(</span><span class="n">NoArgumentLogicalType</span><span class="p">[</span><span class="n">decimal</span><span class="o">.</span><span class="n">Decimal</span><span class="p">,</span> <span class="nb">bytes</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""A logical type for decimal objects handling values consistent with that</span> |
| <span class="sd"> encoded by ``BigDecimalCoder`` in the Java SDK.</span> |
| <span class="sd"> """</span> |
| <div class="viewcode-block" id="DecimalLogicalType.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.DecimalLogicalType.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">common_urns</span><span class="o">.</span><span class="n">decimal</span><span class="o">.</span><span class="n">urn</span></div> |
| |
| <div class="viewcode-block" id="DecimalLogicalType.representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.DecimalLogicalType.representation_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">representation_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| <span class="k">return</span> <span class="nb">bytes</span></div> |
| |
| <div class="viewcode-block" id="DecimalLogicalType.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.DecimalLogicalType.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">decimal</span><span class="o">.</span><span class="n">Decimal</span></div> |
| |
| <div class="viewcode-block" id="DecimalLogicalType.to_representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.DecimalLogicalType.to_representation_type">[docs]</a> <span class="k">def</span> <span class="nf">to_representation_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (decimal.Decimal) -> bytes</span> |
| <span class="k">return</span> <span class="nb">str</span><span class="p">(</span><span class="n">value</span><span class="p">)</span><span class="o">.</span><span class="n">encode</span><span class="p">()</span></div> |
| |
| <div class="viewcode-block" id="DecimalLogicalType.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.DecimalLogicalType.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (bytes) -> decimal.Decimal</span> |
| <span class="k">return</span> <span class="n">decimal</span><span class="o">.</span><span class="n">Decimal</span><span class="p">(</span><span class="n">value</span><span class="o">.</span><span class="n">decode</span><span class="p">())</span></div></div> |
| |
| |
| <div class="viewcode-block" id="FixedPrecisionDecimalLogicalType"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedPrecisionDecimalLogicalType">[docs]</a><span class="nd">@LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span> |
| <span class="k">class</span> <span class="nc">FixedPrecisionDecimalLogicalType</span><span class="p">(</span> |
| <span class="n">LogicalType</span><span class="p">[</span><span class="n">decimal</span><span class="o">.</span><span class="n">Decimal</span><span class="p">,</span> |
| <span class="n">DecimalLogicalType</span><span class="p">,</span> |
| <span class="n">FixedPrecisionDecimalArgumentRepresentation</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""A wrapper of DecimalLogicalType that contains the precision value.</span> |
| <span class="sd"> """</span> |
| <span class="k">def</span> <span class="fm">__init__</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">precision</span><span class="o">=-</span><span class="mi">1</span><span class="p">,</span> <span class="n">scale</span><span class="o">=</span><span class="mi">0</span><span class="p">):</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">precision</span> <span class="o">=</span> <span class="n">precision</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">scale</span> <span class="o">=</span> <span class="n">scale</span> |
| |
| <div class="viewcode-block" id="FixedPrecisionDecimalLogicalType.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedPrecisionDecimalLogicalType.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># TODO(https://github.com/apache/beam/issues/23373) promote this URN to</span> |
| <span class="c1"># schema.proto once logical types with argument are fully supported and the</span> |
| <span class="c1"># implementation of this logical type can thus be considered standardized.</span> |
| <span class="k">return</span> <span class="s2">"beam:logical_type:fixed_decimal:v1"</span></div> |
| |
| <div class="viewcode-block" id="FixedPrecisionDecimalLogicalType.representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedPrecisionDecimalLogicalType.representation_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">representation_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="c1"># type: () -> type</span> |
| <span class="k">return</span> <span class="n">DecimalLogicalType</span></div> |
| |
| <div class="viewcode-block" id="FixedPrecisionDecimalLogicalType.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedPrecisionDecimalLogicalType.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">decimal</span><span class="o">.</span><span class="n">Decimal</span></div> |
| |
| <div class="viewcode-block" id="FixedPrecisionDecimalLogicalType.to_representation_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedPrecisionDecimalLogicalType.to_representation_type">[docs]</a> <span class="k">def</span> <span class="nf">to_representation_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (decimal.Decimal) -> bytes</span> |
| |
| <span class="k">return</span> <span class="n">DecimalLogicalType</span><span class="p">()</span><span class="o">.</span><span class="n">to_representation_type</span><span class="p">(</span><span class="n">value</span><span class="p">)</span></div> |
| |
| <div class="viewcode-block" id="FixedPrecisionDecimalLogicalType.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedPrecisionDecimalLogicalType.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">):</span> |
| <span class="c1"># type: (bytes) -> decimal.Decimal</span> |
| |
| <span class="k">return</span> <span class="n">DecimalLogicalType</span><span class="p">()</span><span class="o">.</span><span class="n">to_language_type</span><span class="p">(</span><span class="n">value</span><span class="p">)</span></div> |
| |
| <div class="viewcode-block" id="FixedPrecisionDecimalLogicalType.argument_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedPrecisionDecimalLogicalType.argument_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">argument_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">FixedPrecisionDecimalArgumentRepresentation</span></div> |
| |
| <div class="viewcode-block" id="FixedPrecisionDecimalLogicalType.argument"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedPrecisionDecimalLogicalType.argument">[docs]</a> <span class="k">def</span> <span class="nf">argument</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">FixedPrecisionDecimalArgumentRepresentation</span><span class="p">(</span> |
| <span class="n">precision</span><span class="o">=</span><span class="bp">self</span><span class="o">.</span><span class="n">precision</span><span class="p">,</span> <span class="n">scale</span><span class="o">=</span><span class="bp">self</span><span class="o">.</span><span class="n">scale</span><span class="p">)</span></div> |
| |
| <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">_from_typing</span><span class="p">(</span><span class="bp">cls</span><span class="p">,</span> <span class="n">typ</span><span class="p">):</span> |
| <span class="k">return</span> <span class="bp">cls</span><span class="p">()</span></div> |
| |
| |
| <span class="c1"># TODO(yathu,BEAM-10722): Investigate and resolve conflicts in logical type</span> |
| <span class="c1"># registration when more than one logical types sharing the same language type</span> |
| <span class="n">LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span><span class="p">(</span><span class="n">DecimalLogicalType</span><span class="p">)</span> |
| |
| |
| <div class="viewcode-block" id="FixedBytes"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedBytes">[docs]</a><span class="nd">@LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span> |
| <span class="k">class</span> <span class="nc">FixedBytes</span><span class="p">(</span><span class="n">PassThroughLogicalType</span><span class="p">[</span><span class="nb">bytes</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""A logical type for fixed-length bytes."""</span> |
| <div class="viewcode-block" id="FixedBytes.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedBytes.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">common_urns</span><span class="o">.</span><span class="n">fixed_bytes</span><span class="o">.</span><span class="n">urn</span></div> |
| |
| <span class="k">def</span> <span class="fm">__init__</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">length</span><span class="p">:</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">):</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">length</span> <span class="o">=</span> <span class="n">length</span> |
| |
| <div class="viewcode-block" id="FixedBytes.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedBytes.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">)</span> <span class="o">-></span> <span class="nb">type</span><span class="p">:</span> |
| <span class="k">return</span> <span class="nb">bytes</span></div> |
| |
| <div class="viewcode-block" id="FixedBytes.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedBytes.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">:</span> <span class="nb">bytes</span><span class="p">):</span> |
| <span class="n">length</span> <span class="o">=</span> <span class="nb">len</span><span class="p">(</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">length</span> <span class="o">></span> <span class="bp">self</span><span class="o">.</span><span class="n">length</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"value length </span><span class="si">{}</span><span class="s2"> > allowed length </span><span class="si">{}</span><span class="s2">"</span><span class="o">.</span><span class="n">format</span><span class="p">(</span><span class="n">length</span><span class="p">,</span> <span class="bp">self</span><span class="o">.</span><span class="n">length</span><span class="p">))</span> |
| <span class="k">elif</span> <span class="n">length</span> <span class="o"><</span> <span class="bp">self</span><span class="o">.</span><span class="n">length</span><span class="p">:</span> |
| <span class="c1"># padding at the end</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">value</span> <span class="o">+</span> <span class="sa">b</span><span class="s1">'</span><span class="se">\0</span><span class="s1">'</span> <span class="o">*</span> <span class="p">(</span><span class="bp">self</span><span class="o">.</span><span class="n">length</span> <span class="o">-</span> <span class="n">length</span><span class="p">)</span> |
| |
| <span class="k">return</span> <span class="n">value</span></div> |
| |
| <div class="viewcode-block" id="FixedBytes.argument_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedBytes.argument_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">argument_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span></div> |
| |
| <div class="viewcode-block" id="FixedBytes.argument"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedBytes.argument">[docs]</a> <span class="k">def</span> <span class="nf">argument</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="k">return</span> <span class="bp">self</span><span class="o">.</span><span class="n">length</span></div></div> |
| |
| |
| <div class="viewcode-block" id="VariableBytes"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableBytes">[docs]</a><span class="nd">@LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span> |
| <span class="k">class</span> <span class="nc">VariableBytes</span><span class="p">(</span><span class="n">PassThroughLogicalType</span><span class="p">[</span><span class="nb">bytes</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""A logical type for variable-length bytes with specified maximum length."""</span> |
| <div class="viewcode-block" id="VariableBytes.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableBytes.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">common_urns</span><span class="o">.</span><span class="n">var_bytes</span><span class="o">.</span><span class="n">urn</span></div> |
| |
| <span class="k">def</span> <span class="fm">__init__</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">max_length</span><span class="p">:</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span> <span class="o">=</span> <span class="n">np</span><span class="o">.</span><span class="n">iinfo</span><span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">)</span><span class="o">.</span><span class="n">max</span><span class="p">):</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">max_length</span> <span class="o">=</span> <span class="n">max_length</span> |
| |
| <div class="viewcode-block" id="VariableBytes.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableBytes.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">)</span> <span class="o">-></span> <span class="nb">type</span><span class="p">:</span> |
| <span class="k">return</span> <span class="nb">bytes</span></div> |
| |
| <div class="viewcode-block" id="VariableBytes.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableBytes.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">:</span> <span class="nb">bytes</span><span class="p">):</span> |
| <span class="n">length</span> <span class="o">=</span> <span class="nb">len</span><span class="p">(</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">length</span> <span class="o">></span> <span class="bp">self</span><span class="o">.</span><span class="n">max_length</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"value length </span><span class="si">{}</span><span class="s2"> > allowed length </span><span class="si">{}</span><span class="s2">"</span><span class="o">.</span><span class="n">format</span><span class="p">(</span><span class="n">length</span><span class="p">,</span> <span class="bp">self</span><span class="o">.</span><span class="n">max_length</span><span class="p">))</span> |
| |
| <span class="k">return</span> <span class="n">value</span></div> |
| |
| <div class="viewcode-block" id="VariableBytes.argument_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableBytes.argument_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">argument_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span></div> |
| |
| <div class="viewcode-block" id="VariableBytes.argument"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableBytes.argument">[docs]</a> <span class="k">def</span> <span class="nf">argument</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="k">return</span> <span class="bp">self</span><span class="o">.</span><span class="n">max_length</span></div></div> |
| |
| |
| <div class="viewcode-block" id="FixedString"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedString">[docs]</a><span class="nd">@LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span> |
| <span class="k">class</span> <span class="nc">FixedString</span><span class="p">(</span><span class="n">PassThroughLogicalType</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""A logical type for fixed-length string."""</span> |
| <div class="viewcode-block" id="FixedString.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedString.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">common_urns</span><span class="o">.</span><span class="n">fixed_char</span><span class="o">.</span><span class="n">urn</span></div> |
| |
| <span class="k">def</span> <span class="fm">__init__</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">length</span><span class="p">:</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">):</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">length</span> <span class="o">=</span> <span class="n">length</span> |
| |
| <div class="viewcode-block" id="FixedString.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedString.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">)</span> <span class="o">-></span> <span class="nb">type</span><span class="p">:</span> |
| <span class="k">return</span> <span class="nb">str</span></div> |
| |
| <div class="viewcode-block" id="FixedString.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedString.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">:</span> <span class="nb">str</span><span class="p">):</span> |
| <span class="n">length</span> <span class="o">=</span> <span class="nb">len</span><span class="p">(</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">length</span> <span class="o">></span> <span class="bp">self</span><span class="o">.</span><span class="n">length</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"value length </span><span class="si">{}</span><span class="s2"> > allowed length </span><span class="si">{}</span><span class="s2">"</span><span class="o">.</span><span class="n">format</span><span class="p">(</span><span class="n">length</span><span class="p">,</span> <span class="bp">self</span><span class="o">.</span><span class="n">length</span><span class="p">))</span> |
| <span class="k">elif</span> <span class="n">length</span> <span class="o"><</span> <span class="bp">self</span><span class="o">.</span><span class="n">length</span><span class="p">:</span> |
| <span class="c1"># padding at the end</span> |
| <span class="n">value</span> <span class="o">=</span> <span class="n">value</span> <span class="o">+</span> <span class="s1">' '</span> <span class="o">*</span> <span class="p">(</span><span class="bp">self</span><span class="o">.</span><span class="n">length</span> <span class="o">-</span> <span class="n">length</span><span class="p">)</span> |
| |
| <span class="k">return</span> <span class="n">value</span></div> |
| |
| <div class="viewcode-block" id="FixedString.argument_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedString.argument_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">argument_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span></div> |
| |
| <div class="viewcode-block" id="FixedString.argument"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.FixedString.argument">[docs]</a> <span class="k">def</span> <span class="nf">argument</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="k">return</span> <span class="bp">self</span><span class="o">.</span><span class="n">length</span></div></div> |
| |
| |
| <div class="viewcode-block" id="VariableString"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableString">[docs]</a><span class="nd">@LogicalType</span><span class="o">.</span><span class="n">register_logical_type</span> |
| <span class="k">class</span> <span class="nc">VariableString</span><span class="p">(</span><span class="n">PassThroughLogicalType</span><span class="p">[</span><span class="nb">str</span><span class="p">,</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">]):</span> |
| <span class="w"> </span><span class="sd">"""A logical type for variable-length string with specified maximum length."""</span> |
| <div class="viewcode-block" id="VariableString.urn"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableString.urn">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">urn</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">common_urns</span><span class="o">.</span><span class="n">var_char</span><span class="o">.</span><span class="n">urn</span></div> |
| |
| <span class="k">def</span> <span class="fm">__init__</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">max_length</span><span class="p">:</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span> <span class="o">=</span> <span class="n">np</span><span class="o">.</span><span class="n">iinfo</span><span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">int32</span><span class="p">)</span><span class="o">.</span><span class="n">max</span><span class="p">):</span> |
| <span class="bp">self</span><span class="o">.</span><span class="n">max_length</span> <span class="o">=</span> <span class="n">max_length</span> |
| |
| <div class="viewcode-block" id="VariableString.language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableString.language_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">language_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">)</span> <span class="o">-></span> <span class="nb">type</span><span class="p">:</span> |
| <span class="k">return</span> <span class="nb">str</span></div> |
| |
| <div class="viewcode-block" id="VariableString.to_language_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableString.to_language_type">[docs]</a> <span class="k">def</span> <span class="nf">to_language_type</span><span class="p">(</span><span class="bp">self</span><span class="p">,</span> <span class="n">value</span><span class="p">:</span> <span class="nb">str</span><span class="p">):</span> |
| <span class="n">length</span> <span class="o">=</span> <span class="nb">len</span><span class="p">(</span><span class="n">value</span><span class="p">)</span> |
| <span class="k">if</span> <span class="n">length</span> <span class="o">></span> <span class="bp">self</span><span class="o">.</span><span class="n">max_length</span><span class="p">:</span> |
| <span class="k">raise</span> <span class="ne">ValueError</span><span class="p">(</span> |
| <span class="s2">"value length </span><span class="si">{}</span><span class="s2"> > allowed length </span><span class="si">{}</span><span class="s2">"</span><span class="o">.</span><span class="n">format</span><span class="p">(</span><span class="n">length</span><span class="p">,</span> <span class="bp">self</span><span class="o">.</span><span class="n">max_length</span><span class="p">))</span> |
| |
| <span class="k">return</span> <span class="n">value</span></div> |
| |
| <div class="viewcode-block" id="VariableString.argument_type"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableString.argument_type">[docs]</a> <span class="nd">@classmethod</span> |
| <span class="k">def</span> <span class="nf">argument_type</span><span class="p">(</span><span class="bp">cls</span><span class="p">):</span> |
| <span class="k">return</span> <span class="n">np</span><span class="o">.</span><span class="n">int32</span></div> |
| |
| <div class="viewcode-block" id="VariableString.argument"><a class="viewcode-back" href="../../../apache_beam.typehints.schemas.html#apache_beam.typehints.schemas.VariableString.argument">[docs]</a> <span class="k">def</span> <span class="nf">argument</span><span class="p">(</span><span class="bp">self</span><span class="p">):</span> |
| <span class="k">return</span> <span class="bp">self</span><span class="o">.</span><span class="n">max_length</span></div></div> |
| </pre></div> |
| |
| </div> |
| |
| </div> |
| <footer> |
| |
| |
| <hr/> |
| |
| <div role="contentinfo"> |
| <p> |
| © Copyright |
| |
| </p> |
| </div> |
| Built with <a href="http://sphinx-doc.org/">Sphinx</a> using a <a href="https://github.com/rtfd/sphinx_rtd_theme">theme</a> provided by <a href="https://readthedocs.org">Read the Docs</a>. |
| |
| </footer> |
| |
| </div> |
| </div> |
| |
| </section> |
| |
| </div> |
| |
| |
| |
| <script type="text/javascript"> |
| jQuery(function () { |
| SphinxRtdTheme.Navigation.enable(true); |
| }); |
| </script> |
| |
| |
| |
| |
| |
| |
| </body> |
| </html> |