You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by se...@apache.org on 2016/04/01 18:44:32 UTC

[9/9] flink git commit: [docs] Add "concepts" documentation page

[docs] Add "concepts" documentation page


Project: http://git-wip-us.apache.org/repos/asf/flink/repo
Commit: http://git-wip-us.apache.org/repos/asf/flink/commit/9aa207c0
Tree: http://git-wip-us.apache.org/repos/asf/flink/tree/9aa207c0
Diff: http://git-wip-us.apache.org/repos/asf/flink/diff/9aa207c0

Branch: refs/heads/master
Commit: 9aa207c073f878d711ca8107627c1f9b2ac17507
Parents: 0331882
Author: Stephan Ewen <se...@apache.org>
Authored: Thu Mar 31 19:25:35 2016 +0200
Committer: Stephan Ewen <se...@apache.org>
Committed: Fri Apr 1 18:39:03 2016 +0200

----------------------------------------------------------------------
 docs/concepts/concepts.md                       | 246 ++++++
 docs/concepts/fig/checkpoints.svg               | 249 ++++++
 .../fig/event_ingestion_processing_time.svg     | 375 ++++++++++
 docs/concepts/fig/parallel_dataflow.svg         | 487 ++++++++++++
 docs/concepts/fig/processes.svg                 | 749 +++++++++++++++++++
 docs/concepts/fig/program_dataflow.svg          | 546 ++++++++++++++
 docs/concepts/fig/slot_sharing.svg              | 721 ++++++++++++++++++
 docs/concepts/fig/state_partitioning.svg        | 291 +++++++
 docs/concepts/fig/tasks_chains.svg              | 463 ++++++++++++
 docs/concepts/fig/tasks_slots.svg               | 395 ++++++++++
 docs/concepts/fig/windows.svg                   | 193 +++++
 docs/page/css/flink.css                         |   9 +
 12 files changed, 4724 insertions(+)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/flink/blob/9aa207c0/docs/concepts/concepts.md
----------------------------------------------------------------------
diff --git a/docs/concepts/concepts.md b/docs/concepts/concepts.md
new file mode 100644
index 0000000..1e8e247
--- /dev/null
+++ b/docs/concepts/concepts.md
@@ -0,0 +1,246 @@
+---
+title: "Concepts"
+---
+
+<!--
+Licensed to the Apache Software Foundation (ASF) under one
+or more contributor license agreements.  See the NOTICE file
+distributed with this work for additional information
+regarding copyright ownership.  The ASF licenses this file
+to you under the Apache License, Version 2.0 (the
+"License"); you may not use this file except in compliance
+with the License.  You may obtain a copy of the License at
+
+  http://www.apache.org/licenses/LICENSE-2.0
+
+Unless required by applicable law or agreed to in writing,
+software distributed under the License is distributed on an
+"AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+KIND, either express or implied.  See the License for the
+specific language governing permissions and limitations
+under the License.
+-->
+
+<script type="text/javascript">
+$( document ).ready(function() {
+  // Handler for .ready() called.
+  $('.ga-track').click( function () {
+    console.log("tracking " + $(this).attr('id'))
+    // we just use the element id for tracking with google analytics
+    ga('send', 'event', 'button', 'click', $(this).attr('id'));
+  });
+
+});
+</script>
+
+* This will be replaced by the TOC
+{:toc}
+
+
+## Programs and Dataflows
+
+The basic building blocks of Flink programs are **streams** and **transformations** (note that a DataSet is internally
+also a stream). A *stream* is an intermediate result, and a *transformation* is an operation that takes one or more streams
+as input, and computes one or more result streams from them.
+
+When executed, Flink programs are mapped to **streaming dataflows**, consisting of **streams** and transformation **operators**.
+Each dataflow starts with one or more **sources** and ends in one or more **sinks**. The dataflows may resemble
+arbitrary **directed acyclic graphs** *(DAGs)*. (Special forms of cycle is permitted via *iteration* constructs, we
+omit this here for simplicity).
+
+In most cases, there is a one-to-one correspondence between the transformations in the programs and the operators
+in the dataflow. Sometimes, however, one transformation may consist of multiple transformation operators.
+
+<img src="fig/program_dataflow.svg" alt="A DataStream program, and its dataflow." class="offset" width="80%" />
+
+
+**Parallel Dataflows**
+
+Programs in Flink are inherently parallel and distributed. *Streams* are split into **stream partitions** and 
+*operators* are split into **operator subtasks**. The operator subtasks execute independently from each other,
+in different threads and on different machines or containers.
+
+The number of operator subtasks is the **parallelism** of that particular operator. The parallelism of a stream
+is always that of its producing operator. Different operators of the program may have a different parallelism.
+
+<img src="fig/parallel_dataflow.svg" alt="A parallel dataflow" class="offset" width="80%" />
+
+Streams can transport data between two operators in a *one-to-one* (or *forwarding*) pattern, or in a *redistributing* pattern:
+
+  - **One-to-one** streams (for example between the *source* and the *map()* operators) preserves partitioning and order of
+    elements. That means that subtask[1] of the *map()* operator will see the same elements in the same order, as they
+    were produced by subtask[1] of the *source* operator.
+
+  - **Redistributing** streams (between *map()* and *keyBy/window*, as well as between *keyBy/window* and *sink*) change
+    the partitioning of streams. Each *stream partition* splits itself up and sends data to different target subtasks,
+    depending on the selected transformation. Examples are *keyBy()* (re-partitions by hash code), *broadcast()*, or
+    *rebalance()* (random redistribution). 
+    In a *redistributing* exchange, order among elements is only preserved for each pair of sending- and receiving
+    task (for example subtask[1] of *map()* and subtask[2] of *keyBy/window*).
+
+
+**Tasks & Operator Chains**
+
+For the distributed execution, Flink *chains* operator subtasks together into *tasks*. Each task is executed by one thread.
+Chaining operators together into tasks is a useful optimization: it reduces the overhead of thread-to-thread
+handover and buffering, and increases overall throughput while decreasing latency.
+The chaining behavior can be configured in the APIs.
+
+The sample dataflow in the figure below is executed with five subtasks, and hence with five parallel threads.
+
+<img src="fig/tasks_chains.svg" alt="Operator chaining into Tasks" class="offset" width="80%" />
+
+
+## Distributed Execution
+
+**Master, Worker, Client**
+
+The Flink runtime consists of two types of processes:
+
+  - The **master** processes (also called *JobManagers*) coordinate the distributed execution. They schedule tasks, coordinate
+    checkpoints, coordinate recovery on failures, etc.
+
+    There is always at least one master process. A high-availability setup will have multiple master processes, out of
+    which one is always the *leader*, and the others are *standby*.
+
+  - The **worker** processes (also called *TaskManagers*) execute the *tasks* (or more specifically, the subtasks) of a dataflow,
+    and buffer and exchange the data *streams*.
+     
+    There must always be at least one worker process.
+
+The master and worker processes can be started in an arbitrary fashion: Directly on the machines, via containers, or via
+resource frameworks like YARN. Workers connect to masters, announcing themselves as available, and get work assigned.
+
+The **client** is not part of the runtime and program execution, but is used to prepare and send to dataflow to the master.
+After that, the client can disconnect, or stay connected to receive progress reports. The client runs either as part of the
+Java/Scala program that triggers the execution, or in the command line process `./bin/flink run ...`.
+
+<img src="fig/processes.svg" alt="The processes involved in executing a Flink dataflow" class="offset" width="80%" />
+
+
+**Workers, Slots, Resources**
+
+Each worker (TaskManager) is a *JVM process*, and may execute one or more subtasks in separate threads.
+To control how many tasks a worker accepts, a worker has so called **task slots** (at least one).
+
+Each *task slot* is a fix subset of resources of the TaskManager. A TaskManager with three slots, for example,
+will dedicate 1/3 of its managed memory to each slot. Slotting the resources means that a subtask will not
+compete with subtasks from other jobs for managed memory, but that the subtask a certain amount of reserved
+managed memory. Note that no CPU isolation happens here, slots currently only separate managed memory of tasks.
+
+Adjusting the number of task slots thus allows users to define how subtasks are isolated against each other.
+Having one slot per TaskManager means each task group runs in a separate JVM (which can be started in a
+separate container, for example). Having multiple slots
+means more subtasks share the same JVM. Tasks in the same JVM share TCP connections (via multiplexing) and
+heartbeats messages, or may shared data sets and data structures, thus reducing the per-task overhead.
+
+<img src="fig/tasks_slots.svg" alt="A TaskManager with Task Slots and Tasks" class="offset" width="80%" />
+
+By default, Flink allows subtasks to share slots, if they are subtasks of different tasks, but from the same
+job. The result is that one slot may hold an entire pipeline of the job. Allowing this *slot sharing*
+has two main benefits:
+
+  - A Flink cluster needs exactly as many tasks slots, as the highest parallelism used in the job.
+    No need to calculate how many tasks (with varying parallelism) a program contains in total.
+
+  - It is easier to get better resource utilization. Without slot sharing, the non-intensive
+    *source/map()* subtasks would block as many resources as the resource intensive *window* subtasks.
+    With slot sharing, increasing the base parallelism from two to six yields full utilization of the
+    slotted resources, while still making sure that each TaskManager gets only a fair share of the
+    heavy subtasks.
+
+The slot sharing behavior can be controlled in the APIs, to prevent sharing where it is undesirable.
+The mechanism for that are the *resource groups*, which define what (sub)tasks may share slots.
+
+As a rule-of-thumb, a good default number of task slots would be the number of CPU cores.
+With hyper threading, each slot then takes 2 or more hardware thread contexts.
+
+<img src="fig/slot_sharing.svg" alt="TaskManagers with shared Task Slots" class="offset" width="80%" />
+
+
+## Time and Windows
+
+Aggregating events (e.g., counts, sums) work slightly differently on streams than in batch processing.
+For example, it is impossible to first count all elements in the stream and then return the count,
+because streams are in general infinite (unbounded). Instead, aggregates on streams (counts, sums, etc),
+are scoped by **windows**, such as *"count over the last 5 minutes"*, or *"sum of the last 100 elements"*.
+
+Windows can be *time driven* (example: every 30 seconds) or *data driven* (example: every 100 elements).
+One typically distinguishes different types of windows, such as *tumbling windows* (no overlap),
+*sliding windows* (with overlap), and *session windows* (gap of activity).
+
+<img src="fig/windows.svg" alt="Time- and Count Windows" class="offset" width="80%" />
+
+More window examples can be found in this [blog post](https://flink.apache.org/news/2015/12/04/Introducing-windows.html).
+
+
+**Time**
+
+When referring to time in a streaming program (for example to define windows), one can refer to different notions
+of time:
+
+  - **Event Time** is the time when an event was created. It is usually described by a timestamp in the events,
+    for example attached by the producing sensor, or the producing service. Flink accesses event timestamps
+    via [timestamp assigners]({{ site.baseurl }}/apis/streaming/event_timestamps_watermarks.html).
+
+  - **Ingestion time** is the time when an event enters the Flink dataflow at the source operator.
+
+  - **Processing Time** is the local time at each operator that performs a time-based operation.
+
+<img src="fig/event_ingestion_processing_time.svg" alt="Event Time, Ingestion Time, and Processing Time" class="offset" width="80%" />
+
+More details on how to handle time are in the [event time docs]({{ site.baseurl }}/apis/streaming/event_time.html).
+
+
+## State and Fault Tolerance
+
+While many operations in a dataflow simply look at one individual *event at a time* (for example an event parser),
+some operations remember information across individual events (for example window operators).
+These operations are called **stateful**.
+
+The state from stateful operation is maintained in what can be thought of as an embedded key/value store.
+The state is partitioned and distributed strictly together with the streams that are read by the
+stateful operators. Hence, access the key/value state is only possible on *keyed streams*, after a *keyBy()* function,
+and is restricted to the values of the current event's key. Aligning the keys of streams and state
+makes sure that all state updates are local operations, guaranteeing consistency without transaction overhead.
+This alignment also allows Flink to redistribute the state and adjust the stream partitioning transparently.
+
+<img src="fig/state_partitioning.svg" alt="State and Partitioning" class="offset" width="50%" />
+
+**Checkpoints for Fault Tolerance**
+
+Flink implements fault tolerance using a combination of **stream replay** and **checkpoints**. A checkpoint
+defines a consistent point in streams and state from which an streaming dataflow can resume, and maintain consistency
+*(exactly-once processing semantics)*. The events and state update since the last checkpoint are replayed from the input streams.
+
+Checkpoints interval is a means of trading off the overhead of fault tolerance during execution, with the recovery time (the amount
+of events that need to be replayed).
+
+More details on checkpoints and fault tolerance are in the [fault tolerance docs]({{ site.baseurl }}/internals/stream_checkpointing.html/).
+
+<img src="fig/checkpoints.svg" alt="checkpoints and snapshots" class="offset" width="60%" />
+
+**State backends**
+
+The exact data structures in which the key/values indexes are stored depend on the chosen **state backend**. One state backend
+stores data in an in-memory hash map, another state backend uses [RocksDB](http://rocksdb.org) as the key/value index.
+In addition to defining the data structure that holds the state, the state backends also implements the logic to
+take a point-in-time snapshot of the key/value state and store that snapshot as part of a checkpoint.
+
+
+## Batch on Streaming
+
+Flink executes batch programs as a special case of streaming programs, where the streams are bounded (finite number of elements).
+A *DataSet* is treated internally as a stream of data. The concepts above thus apply to batch programs in the
+same way as well as they apply to streaming programs, with minor exceptions:
+
+  - Programs in the DataSet API do not use checkpoints. Recovery happens by fully replaying the streams.
+    That is possible, because inputs are bounded. This pushes the cost more towards the recovery,
+    but makes the regular processing cheaper, because it avoids checkpoints.
+
+  - Stateful operation in the DataSet API use simplified in-memory/out-of-core data structures, rather than
+    key/value indexes.
+
+  - The DataSet API introduces special synchronized (superstep-based) iterations, which are only possible on
+    bounded streams. For details, check out the [iteration docs]({{ site.baseurl }}/apis/batch/iterations.html).
+

http://git-wip-us.apache.org/repos/asf/flink/blob/9aa207c0/docs/concepts/fig/checkpoints.svg
----------------------------------------------------------------------
diff --git a/docs/concepts/fig/checkpoints.svg b/docs/concepts/fig/checkpoints.svg
new file mode 100644
index 0000000..c824296
--- /dev/null
+++ b/docs/concepts/fig/checkpoints.svg
@@ -0,0 +1,249 @@
+<?xml version="1.0" encoding="UTF-8" standalone="no"?>
+<!--
+Licensed to the Apache Software Foundation (ASF) under one
+or more contributor license agreements.  See the NOTICE file
+distributed with this work for additional information
+regarding copyright ownership.  The ASF licenses this file
+to you under the Apache License, Version 2.0 (the
+"License"); you may not use this file except in compliance
+with the License.  You may obtain a copy of the License at
+
+  http://www.apache.org/licenses/LICENSE-2.0
+
+Unless required by applicable law or agreed to in writing,
+software distributed under the License is distributed on an
+"AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+KIND, either express or implied.  See the License for the
+specific language governing permissions and limitations
+under the License.
+-->
+
+<svg
+   xmlns:dc="http://purl.org/dc/elements/1.1/"
+   xmlns:cc="http://creativecommons.org/ns#"
+   xmlns:rdf="http://www.w3.org/1999/02/22-rdf-syntax-ns#"
+   xmlns:svg="http://www.w3.org/2000/svg"
+   xmlns="http://www.w3.org/2000/svg"
+   xmlns:sodipodi="http://sodipodi.sourceforge.net/DTD/sodipodi-0.dtd"
+   xmlns:inkscape="http://www.inkscape.org/namespaces/inkscape"
+   width="481.59604"
+   height="368.51669"
+   id="svg2"
+   version="1.1"
+   inkscape:version="0.48.5 r10040"
+   sodipodi:docname="state_partitioning.svg">
+  <defs
+     id="defs4" />
+  <sodipodi:namedview
+     id="base"
+     pagecolor="#ffffff"
+     bordercolor="#666666"
+     borderopacity="1.0"
+     inkscape:pageopacity="0.0"
+     inkscape:pageshadow="2"
+     inkscape:zoom="2.8"
+     inkscape:cx="354.96251"
+     inkscape:cy="137.95685"
+     inkscape:document-units="px"
+     inkscape:current-layer="layer1"
+     showgrid="false"
+     fit-margin-right="0.5"
+     fit-margin-bottom="0.3"
+     fit-margin-top="0.3"
+     fit-margin-left="0"
+     inkscape:window-width="2560"
+     inkscape:window-height="1418"
+     inkscape:window-x="1592"
+     inkscape:window-y="-8"
+     inkscape:window-maximized="1" />
+  <metadata
+     id="metadata7">
+    <rdf:RDF>
+      <cc:Work
+         rdf:about="">
+        <dc:format>image/svg+xml</dc:format>
+        <dc:type
+           rdf:resource="http://purl.org/dc/dcmitype/StillImage" />
+        <dc:title></dc:title>
+      </cc:Work>
+    </rdf:RDF>
+  </metadata>
+  <g
+     inkscape:label="Layer 1"
+     inkscape:groupmode="layer"
+     id="layer1"
+     transform="translate(-130.78007,-350.87488)">
+    <g
+       id="g3138"
+       transform="translate(116.16121,190.10975)">
+      <path
+         id="path3140"
+         d="m 78.39453,344.07322 0,74.86865 95.01117,0 0,-74.86865 -95.01117,0 z"
+         style="fill:#c5e0b4;fill-opacity:1;fill-rule:evenodd;stroke:none"
+         inkscape:connector-curvature="0" />
+      <text
+         id="text3142"
+         style="font-size:17.55437279px;font-style:normal;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="377.96512"
+         x="106.64163"
+         xml:space="preserve">Task</text>
+      <text
+         id="text3144"
+         style="font-size:17.55437279px;font-style:normal;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="398.97034"
+         x="88.036995"
+         xml:space="preserve">Manager</text>
+      <path
+         id="path3146"
+         d="m 207.48294,344.07322 0,74.86865 95.02992,0 0,-74.86865 -95.02992,0 z"
+         style="fill:#c5e0b4;fill-opacity:1;fill-rule:evenodd;stroke:none"
+         inkscape:connector-curvature="0" />
+      <text
+         id="text3148"
+         style="font-size:17.55437279px;font-style:normal;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="377.96512"
+         x="235.75273"
+         xml:space="preserve">Task</text>
+      <text
+         id="text3150"
+         style="font-size:17.55437279px;font-style:normal;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="398.97034"
+         x="217.1481"
+         xml:space="preserve">Manager</text>
+      <path
+         id="path3152"
+         d="m 336.57135,344.07322 0,74.86865 95.17996,0 0,-74.86865 -95.17996,0 z"
+         style="fill:#c5e0b4;fill-opacity:1;fill-rule:evenodd;stroke:none"
+         inkscape:connector-curvature="0" />
+      <text
+         id="text3154"
+         style="font-size:17.55437279px;font-style:normal;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="377.96512"
+         x="364.86383"
+         xml:space="preserve">Task</text>
+      <text
+         id="text3156"
+         style="font-size:17.55437279px;font-style:normal;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="398.97034"
+         x="346.25919"
+         xml:space="preserve">Manager</text>
+      <path
+         id="path3158"
+         d="m 93.079438,161.06513 0,74.85927 95.179962,0 0,-74.85927 -95.179962,0 z"
+         style="fill:#ffd966;fill-opacity:1;fill-rule:evenodd;stroke:none"
+         inkscape:connector-curvature="0" />
+      <text
+         id="text3160"
+         style="font-size:17.55437279px;font-style:normal;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="194.95898"
+         x="125.94909"
+         xml:space="preserve">Job</text>
+      <text
+         id="text3162"
+         style="font-size:17.55437279px;font-style:normal;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="215.96423"
+         x="102.84333"
+         xml:space="preserve">Manager</text>
+      <text
+         id="text3164"
+         style="font-size:11.2528038px;font-style:italic;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="202.80112"
+         x="33.991787"
+         xml:space="preserve">(master)</text>
+      <text
+         id="text3166"
+         style="font-size:11.2528038px;font-style:italic;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="385.80722"
+         x="13.838635"
+         xml:space="preserve">(workers)</text>
+      <path
+         id="path3168"
+         d="m 106.5828,243.0418 -2.25056,5.53263 0,0 -2.15679,5.53262 0,0 -1.0315,2.7757 0,0 -0.994,2.79444 0.0187,-0.0188 -0.918974,2.79444 0,0 -0.843961,2.8132 0,-0.0188 -0.768941,2.83196 0,-0.0188 -0.675168,2.85071 0,-0.0188 -0.581395,2.86946 0,-0.0188 -0.468867,2.88822 0.01875,-0.0188 -0.356339,2.90698 0,-0.0188 -0.225056,2.94448 0,-0.0375 -0.07502,2.96324 0,-0.0187 0.07502,3.00075 0,-0.0375 0.225056,3.0195 0,-0.0188 0.375093,3.05702 -0.01875,-0.0375 0.52513,3.09452 0,-0.0188 0.637659,3.11328 0,-0.0188 0.768942,3.13203 0,-0.0188 0.881469,3.15078 -0.01875,-0.0188 0.975243,3.16954 0,-0.0188 1.069009,3.20705 -0.0187,-0.0187 1.16279,3.20705 0,0 1.21905,3.20705 -0.0187,0 1.27532,3.2258 0,0 1.33158,3.24456 0,-0.0188 2.75693,6.50788 0,0 2.34434,5.40134 -1.14404,0.48762 -2.34433,-5.38259 -2.77569,-6.52662 -1.31283,-3.24456 -1.29407,-3.24456 -1.21906,-3.2258 -1.162782,-3.22581 -1.050262,-3.20705 -0.993997,-3.18829 -0.88147,-3.16954 -0.768941,-3.15079 -0.656414,-3.13203 -0.525131,-3.11327
  -0.375093,-3.09452 -0.225056,-3.05701 -0.05626,-3.03826 0.07502,-2.98199 0.225056,-2.982 0.337585,-2.94448 0.487621,-2.92573 0.581395,-2.88822 0.675168,-2.86946 0.787696,-2.85071 0.843961,-2.83196 0.918979,-2.8132 0.993997,-2.8132 1.031508,-2.79445 2.17554,-5.55138 2.25056,-5.53263 z m 5.30757,86.5153 -1.14403,9.69617 -7.87696,-5.77644 c -0.28132,-0.2063 -0.33759,-0.60015 -0.13129,-0.86272 0.20631,-0.28132 0.60015,-0.33758 0.88147,-0.15003 l 6.9955,5.13878 -0.97525,0.43135 1.01276,-8.62715 c 0.0375,-0.33758 0.35634,-0.58139 0.69392,-0.54388 0.33758,0.0375 0.58139,0.35634 0.54388,0.69392 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3170"
+         d="m 115.3975,337.34029 1.70667,-6.07651 0,0 1.65041,-6.07652 0,0.0188 1.53789,-6.07652 0,0.0188 1.36909,-6.02025 0,0 0.60015,-3.00075 0,0.0188 0.54388,-3.00075 0,0.0188 0.46887,-3.00075 0,0.0188 0.39385,-2.96324 0,0 0.30007,-2.94448 0,0.0188 0.22506,-2.96324 0,0.0375 0.13128,-2.92573 0,0.0188 0.0188,-2.92573 0,0.0375 -0.0938,-2.90697 0,0.0188 -0.2063,-2.86946 0,0.0188 -0.28132,-2.86946 0,0.0188 -0.39385,-2.85071 0.0188,0.0187 -0.48762,-2.85071 0,0.0188 -0.54389,-2.8132 0,0.0188 -0.6189,-2.8132 0.0187,0 -0.67517,-2.8132 0,0.0188 -1.53788,-5.57014 0.0188,0.0188 -1.68792,-5.57014 0,0.0188 -1.80045,-5.55138 0,0 -1.46287,-4.35109 1.18155,-0.4126 1.46286,4.36984 1.80045,5.55138 1.68792,5.58889 1.53788,5.5889 0.67517,2.83195 0.63766,2.83196 0.54389,2.83195 0.46886,2.85071 0.39385,2.88822 0.30008,2.88822 0.18754,2.88822 0.0938,2.92573 -0.0188,2.94448 -0.11253,2.94449 -0.22505,2.96324 -0.31883,2.98199 -0.39385,2.98199 -0.46887,3.0195 -0.54388,3.00075 -0.61891,3.0195 -1.36909,6.0390
 1 -1.53788,6.07651 -1.65041,6.09527 -1.70668,6.07651 z m -2.56939,-83.25199 1.96924,-9.56488 7.35183,6.43285 c 0.26257,0.22506 0.28132,0.6189 0.0563,0.88147 -0.22505,0.26256 -0.6189,0.28132 -0.88147,0.0563 l -6.52662,-5.72017 1.01275,-0.33759 -1.76294,8.49587 c -0.075,0.33758 -0.39385,0.56264 -0.73143,0.48762 -0.33758,-0.075 -0.56264,-0.39385 -0.48762,-0.73143 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3172"
+         d="m 127.0629,243.41689 10.12753,7.78319 5.02625,3.88222 4.98874,3.84471 4.91373,3.82595 4.85746,3.76969 4.76368,3.73218 4.65116,3.67591 4.53863,3.6009 4.40735,3.54463 4.27607,3.46962 4.10727,3.35708 3.91973,3.28207 3.73218,3.16954 3.52587,3.05701 1.68792,1.50038 1.61291,1.44411 1.57539,1.42535 1.50037,1.38785 2.86947,2.70067 2.62565,2.58814 2.47562,2.47562 2.26931,2.38184 2.11928,2.28807 1.96924,2.23181 1.83796,2.11928 1.72543,2.08177 1.6129,2.0255 1.51913,1.96924 1.44411,1.95049 1.38785,1.87547 1.36909,1.89422 1.89422,2.68192 -1.03151,0.71267 -1.89422,-2.68191 0,0 -1.35034,-1.87547 0.0188,0.0188 -1.38785,-1.89422 0,0 -1.44411,-1.93173 0,0.0188 -1.51913,-1.96924 0,0.0188 -1.59414,-2.02551 0,0.0188 -1.70668,-2.08177 0,0.0187 -1.8192,-2.13803 0,0.0188 -1.95049,-2.21305 0,0 -2.10052,-2.26932 0,0 -2.26932,-2.38184 0,0 -2.45686,-2.45687 0.0188,0 -2.64441,-2.56939 0.0188,0 -2.85071,-2.68191 0,0 -1.50037,-1.38785 0,0 -1.55664,-1.4066 0,0 -1.63166,-1.46286 0,0 -1.66916,-1.48162 0,
 0.0188 -3.52588,-3.05701 0,0 -3.71342,-3.16954 0,0 -3.91973,-3.26331 0,0 -4.08852,-3.37584 0,0 -4.27607,-3.45086 0.0188,0 -4.40735,-3.54464 0,0.0188 -4.53863,-3.61965 -4.65116,-3.67592 0,0 -4.76368,-3.73218 0,0 -4.83871,-3.76969 -4.93248,-3.82595 -4.96999,-3.84471 -5.02625,-3.86346 -10.12752,-7.80195 z m 100.69384,82.6706 0.84396,9.71492 -8.88971,-4.05101 c -0.30008,-0.15004 -0.45012,-0.50638 -0.30008,-0.82521 0.13128,-0.31883 0.50638,-0.45011 0.82521,-0.31883 l 7.89571,3.61965 -0.88147,0.61891 -0.75018,-8.64591 c -0.0188,-0.35633 0.22505,-0.65641 0.58139,-0.69392 0.33759,-0.0187 0.63766,0.22506 0.67517,0.5814 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3174"
+         d="m 242.72297,335.48358 -4.57614,-8.25206 0,0 -4.5949,-8.15828 0,0 -2.32558,-4.03226 0,0 -2.36308,-3.97599 0,0.0188 -2.38185,-3.91973 0.0188,0 -2.41936,-3.84471 0.0188,0 -2.45686,-3.76968 0,0.0188 -2.47562,-3.67592 0,0 -2.53188,-3.58214 0,0.0188 -2.56939,-3.46962 0,0.0188 -2.64441,-3.35709 0.0188,0.0188 -2.70068,-3.22581 0.0188,0.0188 -2.75694,-3.07577 0.0188,0.0188 -2.83195,-2.92572 0,0.0188 -2.90697,-2.77569 0.0188,0 -2.96324,-2.62566 0.0188,0.0188 -3.03826,-2.47562 0.0188,0.0188 -3.09452,-2.36309 0.0188,0.0188 -3.15079,-2.2318 0.0188,0 -3.1883,-2.11928 0,0.0188 -3.24456,-2.0255 0.0188,0 -3.28207,-1.93173 0.0188,0 -3.33834,-1.85671 0.0188,0 -3.35709,-1.7817 0.0188,0.0188 -3.3946,-1.72543 0.0188,0 -3.41335,-1.68792 0.0187,0.0187 -6.86421,-3.24456 0.0188,0 -5.77644,-2.64441 0.52513,-1.14403 5.77644,2.64441 6.84545,3.24456 3.41335,1.68792 3.3946,1.72543 3.35708,1.80044 3.35709,1.85672 3.28207,1.95048 3.26331,2.02551 3.20705,2.13803 3.16954,2.25056 3.11328,2.36309 3.05701,2.
 49437 2.98199,2.64441 2.92573,2.79445 2.85071,2.94448 2.77569,3.09452 2.70067,3.2258 2.66317,3.37585 2.58814,3.46961 2.55064,3.6009 2.49437,3.69467 2.45686,3.76969 2.4006,3.86346 2.4006,3.91973 2.34433,3.99474 2.34433,4.03225 4.61365,8.17704 4.57614,8.2333 z m -85.57757,-83.60833 -5.60765,-7.97074 9.71492,-0.95649 c 0.33759,-0.0375 0.63766,0.20631 0.67517,0.56264 0.0375,0.33759 -0.22506,0.63766 -0.56264,0.67517 l -8.6459,0.86272 0.45011,-0.994 5.0075,7.10802 c 0.18754,0.28132 0.13128,0.67517 -0.1688,0.88147 -0.28132,0.18755 -0.65641,0.13128 -0.86271,-0.16879 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3176"
+         d="m 341.37254,343.34179 -123.2557,-87.0967 -2.25056,-1.63166 -2.08177,-1.51913 -1.93173,-1.42535 -1.80045,-1.33158 -1.66917,-1.21906 -1.53788,-1.14403 -1.4066,-1.05027 -1.29407,-0.97524 -1.18155,-0.91898 -1.06901,-0.8252 -0.97525,-0.75019 -0.88147,-0.69392 -0.80645,-0.65642 -0.71267,-0.58139 -0.63766,-0.52513 -0.56264,-0.48762 -0.50638,-0.45012 -0.45011,-0.4126 -0.75019,-0.75018 -0.58139,-0.65642 -0.45012,-0.58139 -0.39384,-0.56264 -0.33759,-0.54389 -0.13128,-0.18755 1.06902,-0.67516 0.11252,0.2063 0.33759,0.52513 -0.0188,-0.0188 0.3751,0.54389 -0.0188,-0.0375 0.43136,0.54389 -0.0187,-0.0188 0.54388,0.6189 -0.0188,-0.0375 0.73143,0.73143 -0.0375,-0.0188 0.45011,0.39385 0,0 0.48762,0.45011 0,-0.0188 0.56264,0.48762 -0.0187,0 0.63766,0.52513 0,0 0.71267,0.5814 -0.0188,0 0.7877,0.63765 0,-0.0188 0.88147,0.69392 0,0 0.97524,0.76894 0,0 1.06901,0.82521 0,-0.0188 1.18155,0.90022 0,0 1.29407,0.97524 -0.0187,0 1.4066,1.05026 1.53788,1.14404 0,0 1.65041,1.23781 0,0 1.80045,1.31282 
 1.93173,1.42536 0,0 2.10052,1.51913 2.23181,1.63165 -0.0188,0 123.27446,87.0967 z m -147.31795,-98.61207 -0.48762,-9.73368 8.72092,4.36984 c 0.31883,0.15004 0.45012,0.52513 0.28133,0.84396 -0.15004,0.30008 -0.52514,0.43136 -0.82521,0.28132 l -7.76443,-3.90097 0.90022,-0.58139 0.43136,8.66465 c 0.0188,0.33759 -0.24381,0.63766 -0.60015,0.65642 -0.33759,0.0188 -0.63766,-0.24381 -0.65642,-0.60015 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3178"
+         d="m 200.99382,225.67497 15.21004,6.07652 7.55813,3.05701 7.50187,3.05701 7.42685,3.07576 7.35184,3.09453 7.2393,3.11327 7.10802,3.15079 6.95798,3.15078 6.80795,3.1883 6.63915,3.2258 6.4141,3.26331 3.15079,1.65041 3.07576,1.65041 3.0195,1.66917 2.96324,1.68792 2.88822,1.68792 2.85071,1.68792 2.75694,1.72543 2.70067,1.72543 2.62566,1.74419 2.56939,1.74418 2.47561,1.76294 2.43811,1.78169 2.36309,1.80045 2.28807,1.7817 2.25056,1.8192 2.1943,1.8192 4.20104,3.67592 4.0135,3.69467 3.82596,3.75093 3.67591,3.76969 3.50713,3.78845 3.39459,3.80719 3.28207,3.82596 3.20705,3.86346 3.11327,3.86346 3.07577,3.86346 5.27006,6.77044 -0.97524,0.76894 -5.28882,-6.77043 0,0 -3.05701,-3.86347 0.0188,0 -3.13204,-3.8447 0,0 -3.18829,-3.84471 0.0188,0.0187 -3.28207,-3.82595 0,0 -3.37584,-3.78844 0,0 -3.50713,-3.78845 0.0188,0.0188 -3.65716,-3.75094 0,0 -3.8072,-3.73218 0,0.0188 -3.99475,-3.69467 0.0188,0.0187 -4.20105,-3.65716 0.0188,0 -2.1943,-1.80045 0,0 -2.23181,-1.8192 0.0188,0.0187 -2.28807,-
 1.80045 0,0.0188 -2.34434,-1.78169 0,0 -2.41935,-1.7817 0.0188,0.0188 -2.49438,-1.76294 0.0188,0 -2.55064,-1.74419 0,0 -2.6069,-1.72543 0,0 -2.70067,-1.72543 0.0188,0 -2.75694,-1.70667 0,0.0188 -2.83196,-1.70667 0.0188,0 -2.88822,-1.68792 0,0 -2.96324,-1.66917 0.0187,0 -3.0195,-1.65041 0,0 -3.07576,-1.65041 0,0 -3.13203,-1.65041 0.0188,0 -6.4141,-3.24456 0,0 -6.6204,-3.2258 0.0188,0.0187 -6.80795,-3.18829 0.0188,0 -6.97674,-3.16954 0,0 -7.08927,-3.13203 0,0 -7.2393,-3.09452 0.0188,0 -7.33307,-3.09453 0,0 -7.44561,-3.07576 0,0 -7.48311,-3.05701 -7.55814,-3.05702 0,0 -15.19128,-6.09526 z m 168.3607,101.55655 1.31282,9.67741 -9.0585,-3.6384 c -0.33759,-0.11253 -0.48763,-0.48763 -0.35634,-0.80645 0.13128,-0.31883 0.48762,-0.46887 0.80645,-0.35634 l 8.06451,3.2258 -0.84396,0.67517 -1.16279,-8.6084 c -0.0563,-0.33758 0.18754,-0.65641 0.52513,-0.71267 0.35634,-0.0375 0.67517,0.2063 0.71268,0.54388 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3180"
+         d="m 270.9675,241.59769 -2.58814,-1.33158 -4.8012,5.70142 -0.97524,-0.48762 4.8012,-5.72018 -2.6069,-1.31283 0.60015,-0.73143 6.18904,3.15079 z m 2.79445,3.90097 -0.0375,-0.0375 c -0.11253,-0.0938 -0.22505,-0.16879 -0.33758,-0.24381 -0.0938,-0.075 -0.22506,-0.15004 -0.39385,-0.24381 -0.30008,-0.15004 -0.63766,-0.22506 -1.01275,-0.24381 -0.3751,-0.0188 -0.75019,0 -1.12528,0.0563 l -2.86947,3.43211 -0.91898,-0.46887 4.03226,-4.8387 0.91898,0.46886 -0.5814,0.71268 c 0.5814,-0.075 1.05026,-0.0938 1.42536,-0.0563 0.35634,0.0375 0.67516,0.11252 0.95648,0.26256 0.15004,0.075 0.26257,0.15004 0.33759,0.18755 0.075,0.0375 0.16879,0.11253 0.28132,0.2063 z m 4.0135,-1.53788 -0.71268,0.84396 -1.05026,-0.54389 0.71268,-0.84396 z m -1.46286,1.6129 -4.05101,4.81995 -0.91898,-0.46887 4.05101,-4.81995 z m 6.30157,3.20705 -3.6009,4.29482 c -0.67517,0.80645 -1.36909,1.31282 -2.08177,1.50037 -0.71268,0.16879 -1.50037,0.0375 -2.38184,-0.4126 -0.28132,-0.13128 -0.54389,-0.30008 -0.80645,-0.48762 
 -0.26257,-0.18755 -0.50638,-0.3751 -0.71268,-0.5814 l 0.69392,-0.80645 0.0375,0.0375 c 0.15004,0.16879 0.35634,0.37509 0.61891,0.60015 0.28132,0.24381 0.54388,0.43136 0.8252,0.56264 0.30008,0.15004 0.5814,0.26257 0.84396,0.28132 0.26257,0.0375 0.48762,0.0188 0.71268,-0.0563 0.2063,-0.0563 0.39385,-0.1688 0.58139,-0.30008 0.1688,-0.13128 0.33759,-0.30007 0.50638,-0.50637 l 0.35634,-0.43136 c -0.45011,0.0563 -0.82521,0.075 -1.12528,0.0563 -0.30008,-0.0187 -0.65642,-0.13128 -1.05026,-0.31883 -0.52513,-0.28132 -0.86272,-0.65641 -0.994,-1.12528 -0.13128,-0.48762 -0.0563,-1.01275 0.24381,-1.59414 0.24381,-0.48763 0.56264,-0.90023 0.93773,-1.27532 0.3751,-0.35634 0.80645,-0.63766 1.25657,-0.84396 0.43135,-0.2063 0.90022,-0.30008 1.38784,-0.31883 0.48762,0 0.93774,0.0938 1.36909,0.31883 0.31883,0.16879 0.5814,0.33758 0.7877,0.54388 0.2063,0.18755 0.37509,0.39385 0.50638,0.60015 l 0.22505,-0.16879 z m -1.65041,0.39385 c -0.15004,-0.20631 -0.31883,-0.39385 -0.50638,-0.5814 -0.18755,-0.16879 -
 0.39385,-0.30007 -0.61891,-0.43136 -0.33758,-0.15003 -0.65641,-0.22505 -0.99399,-0.22505 -0.33759,0.0188 -0.67517,0.0938 -0.994,0.26256 -0.30008,0.15004 -0.5814,0.3751 -0.86272,0.65642 -0.26256,0.30007 -0.48762,0.60015 -0.65641,0.95648 -0.2063,0.3751 -0.26256,0.71268 -0.2063,1.01276 0.075,0.30007 0.30007,0.56264 0.69392,0.75018 0.26257,0.15004 0.5814,0.22506 0.91898,0.26257 0.33759,0.0188 0.67517,0.0188 1.03151,-0.0188 z m 7.93322,2.8132 -3.6009,4.29482 c -0.67516,0.80645 -1.36909,1.31282 -2.08176,1.48162 -0.71268,0.18754 -1.50038,0.0563 -2.38185,-0.39385 -0.28132,-0.15004 -0.54388,-0.30008 -0.80645,-0.48762 -0.26256,-0.18755 -0.50637,-0.39385 -0.71268,-0.5814 l 0.69393,-0.80645 0.0375,0.0188 c 0.15003,0.18755 0.35633,0.39385 0.6189,0.61891 0.28132,0.22506 0.54389,0.4126 0.82521,0.56264 0.30007,0.15004 0.58139,0.24381 0.84396,0.28132 0.26256,0.0375 0.48762,0.0188 0.71267,-0.0563 0.20631,-0.0563 0.39385,-0.1688 0.5814,-0.30008 0.16879,-0.15004 0.33758,-0.31883 0.50637,-0.50638 l 0.35
 634,-0.43135 c -0.45011,0.0563 -0.8252,0.075 -1.12528,0.0563 -0.30007,-0.0188 -0.65641,-0.13128 -1.05026,-0.31883 -0.52513,-0.28132 -0.86271,-0.65641 -0.994,-1.12528 -0.13128,-0.48762 -0.0563,-1.01275 0.24381,-1.6129 0.24382,-0.46887 0.56264,-0.88147 0.93774,-1.25656 0.37509,-0.35634 0.80645,-0.65642 1.25656,-0.86272 0.43136,-0.18755 0.90023,-0.30007 1.38785,-0.30007 0.48762,-0.0188 0.93773,0.0938 1.36909,0.31883 0.31883,0.15003 0.58139,0.33758 0.78769,0.54388 0.20631,0.18755 0.3751,0.39385 0.50638,0.60015 l 0.22506,-0.18755 z m -1.65041,0.39384 c -0.15004,-0.2063 -0.31883,-0.4126 -0.50638,-0.58139 -0.18754,-0.16879 -0.39384,-0.31883 -0.6189,-0.43136 -0.33758,-0.16879 -0.65641,-0.24381 -0.994,-0.22505 -0.33758,0.0187 -0.67517,0.0938 -0.994,0.26256 -0.30007,0.15004 -0.58139,0.3751 -0.86271,0.65642 -0.26257,0.28132 -0.48762,0.60014 -0.65641,0.93773 -0.20631,0.39385 -0.26257,0.73143 -0.20631,1.03151 0.075,0.30007 0.30008,0.54388 0.69393,0.75018 0.26256,0.13129 0.58139,0.22506 0.91898,0
 .24381 0.33758,0.0375 0.67516,0.0375 1.0315,0 z m 5.58889,4.29482 c 0.0563,-0.075 0.11253,-0.15003 0.15004,-0.2063 0.0375,-0.075 0.075,-0.13128 0.13128,-0.22505 0.16879,-0.33759 0.2063,-0.65642 0.13129,-0.95649 -0.0938,-0.30008 -0.35634,-0.54389 -0.76895,-0.76894 -0.46886,-0.22506 -0.95648,-0.30008 -1.46286,-0.18755 -0.50638,0.0938 -0.93773,0.35634 -1.31283,0.76894 z m -3.60089,2.53189 c -0.75019,-0.39385 -1.25657,-0.86272 -1.51913,-1.44411 -0.24381,-0.5814 -0.18755,-1.2003 0.16879,-1.89423 0.50638,-0.99399 1.25656,-1.70667 2.25056,-2.10052 0.994,-0.39385 1.96924,-0.33758 2.90697,0.13128 0.61891,0.31883 1.01276,0.71268 1.18155,1.18155 0.16879,0.48762 0.11253,0.994 -0.16879,1.55664 -0.0563,0.0938 -0.15004,0.24381 -0.28132,0.45011 -0.13129,0.18754 -0.30008,0.4126 -0.50638,0.65641 l -4.03225,-2.04426 c -0.075,0.075 -0.13129,0.16879 -0.18755,0.26257 -0.0563,0.075 -0.0938,0.16879 -0.13128,0.24381 -0.24381,0.45011 -0.28132,0.90022 -0.13129,1.29407 0.13129,0.4126 0.46887,0.75019 0.994,1.01
 275 0.35634,0.16879 0.75019,0.30008 1.2003,0.35634 0.45011,0.0375 0.86272,0.0563 1.2003,0 l 0.0563,0.0375 -0.71267,0.88147 c -0.18755,-0.0188 -0.35634,-0.0375 -0.50638,-0.0563 -0.16879,-0.0188 -0.35634,-0.0563 -0.5814,-0.11253 -0.22505,-0.0563 -0.43135,-0.11253 -0.60014,-0.15004 -0.1688,-0.0563 -0.3751,-0.15004 -0.60015,-0.26256 z m 10.20254,-0.63766 -0.0563,-0.0188 c -0.11252,-0.0938 -0.22505,-0.18754 -0.31883,-0.26256 -0.11252,-0.075 -0.24381,-0.15004 -0.39384,-0.22506 -0.31883,-0.15004 -0.65642,-0.24381 -1.03151,-0.26257 -0.35634,0 -0.73143,0.0188 -1.10653,0.0563 l -2.86946,3.4321 -0.93774,-0.46886 4.03226,-4.83871 0.93773,0.48762 -0.60015,0.71268 c 0.5814,-0.0938 1.06902,-0.11253 1.42536,-0.075 0.35634,0.0375 0.67517,0.13128 0.95649,0.26257 0.16879,0.0938 0.28132,0.15003 0.33758,0.18754 0.075,0.0563 0.16879,0.11253 0.30008,0.20631 z m 4.31357,8.04575 c -0.91898,-0.46887 -1.51913,-1.08777 -1.80045,-1.83796 -0.28132,-0.73143 -0.2063,-1.55663 0.24381,-2.41935 0.33759,-0.67517 0.768
 95,-1.25656 1.25657,-1.72543 0.50637,-0.48762 1.05026,-0.84396 1.65041,-1.08777 0.60015,-0.24381 1.23781,-0.35634 1.91297,-0.31883 0.67517,0.0375 1.35034,0.22506 2.02551,0.56264 0.4126,0.2063 0.7877,0.46887 1.10653,0.76894 0.31882,0.28132 0.63765,0.63766 0.91897,1.06902 l -0.78769,0.97524 -0.075,-0.0375 c -0.0563,-0.15004 -0.11253,-0.31883 -0.16879,-0.45011 -0.0563,-0.15004 -0.16879,-0.33758 -0.33759,-0.56264 -0.13128,-0.18755 -0.30007,-0.37509 -0.52513,-0.56264 -0.2063,-0.18755 -0.46886,-0.35634 -0.76894,-0.52513 -0.45011,-0.22506 -0.91898,-0.33759 -1.4066,-0.35634 -0.46887,0 -0.93773,0.11253 -1.38785,0.30007 -0.45011,0.20631 -0.86271,0.48763 -1.27531,0.88147 -0.39385,0.41261 -0.73144,0.88147 -1.01276,1.4066 -0.33758,0.67517 -0.43135,1.29408 -0.28132,1.83796 0.1688,0.56264 0.5814,0.994 1.23781,1.33158 0.30008,0.1688 0.61891,0.28132 0.93774,0.35634 0.31883,0.0563 0.60015,0.11253 0.90022,0.11253 0.26257,0 0.50638,0 0.73143,-0.0375 0.22506,-0.0188 0.41261,-0.0375 0.5814,-0.075 l 0.075
 ,0.0375 -0.80645,0.994 c -0.43136,0 -0.90023,-0.0375 -1.4066,-0.13128 -0.52514,-0.075 -1.03151,-0.26257 -1.53789,-0.50638 z m 11.30907,0.28132 c -0.0375,0.075 -0.0938,0.16879 -0.18755,0.30008 -0.075,0.13128 -0.16879,0.22505 -0.24381,0.31883 l -2.62565,3.13203 -0.91898,-0.46887 2.30682,-2.73818 c 0.13129,-0.1688 0.22506,-0.30008 0.31883,-0.41261 0.0938,-0.11252 0.16879,-0.24381 0.22506,-0.37509 0.15004,-0.26257 0.18755,-0.50638 0.11253,-0.71268 -0.0563,-0.2063 -0.26257,-0.4126 -0.61891,-0.58139 -0.26256,-0.13128 -0.56264,-0.2063 -0.93773,-0.22506 -0.35634,-0.0188 -0.73143,0 -1.12528,0.0563 l -3.0195,3.60089 -0.91898,-0.46886 5.64515,-6.73293 0.91898,0.46887 -2.04426,2.4381 c 0.46887,-0.075 0.90023,-0.0938 1.27532,-0.0563 0.37509,0.0188 0.71268,0.11253 1.03151,0.28132 0.48762,0.24381 0.78769,0.54388 0.93773,0.91898 0.15004,0.37509 0.0938,0.78769 -0.13128,1.25656 z m 4.78244,3.54463 c 0.0563,-0.075 0.11253,-0.15003 0.15004,-0.22505 0.0375,-0.0563 0.075,-0.13128 0.11253,-0.2063 0.18754,
 -0.33759 0.22505,-0.67517 0.13128,-0.95649 -0.0938,-0.30008 -0.33759,-0.54389 -0.76894,-0.76894 -0.45011,-0.22506 -0.93774,-0.30008 -1.44411,-0.18755 -0.50638,0.0938 -0.93774,0.35634 -1.31283,0.75019 z m -3.6009,2.53188 c -0.76894,-0.39384 -1.25656,-0.88146 -1.51912,-1.44411 -0.24382,-0.58139 -0.18755,-1.20029 0.15003,-1.89422 0.52513,-0.99399 1.27532,-1.70667 2.26932,-2.10052 0.994,-0.39385 1.96924,-0.35634 2.88822,0.13128 0.6189,0.31883 1.01275,0.71268 1.2003,1.18155 0.16879,0.48762 0.11252,0.99399 -0.1688,1.55663 -0.0563,0.0938 -0.15003,0.24382 -0.28132,0.45012 -0.13128,0.18754 -0.31883,0.4126 -0.52513,0.65641 l -4.0135,-2.04426 c -0.075,0.075 -0.13128,0.16879 -0.18754,0.26257 -0.0563,0.075 -0.11253,0.16879 -0.15004,0.24381 -0.22506,0.45011 -0.28132,0.88147 -0.13128,1.29407 0.15003,0.4126 0.48762,0.75019 0.99399,1.01275 0.35634,0.16879 0.76895,0.30008 1.21906,0.33759 0.45011,0.0563 0.84396,0.075 1.2003,0.0187 l 0.0563,0.0375 -0.71268,0.88147 c -0.18754,-0.0188 -0.35634,-0.0375 -0
 .52513,-0.0563 -0.15003,-0.0188 -0.33758,-0.0563 -0.56264,-0.11253 -0.22505,-0.0563 -0.43135,-0.11253 -0.60015,-0.16879 -0.16879,-0.0375 -0.37509,-0.13129 -0.60015,-0.24382 z m 5.7952,2.94449 c -0.35634,-0.16879 -0.63766,-0.3751 -0.86272,-0.60015 -0.22505,-0.2063 -0.39385,-0.45011 -0.48762,-0.73143 -0.11253,-0.26257 -0.15004,-0.56264 -0.13128,-0.86272 0.0375,-0.31883 0.13128,-0.63766 0.30007,-0.994 0.26257,-0.50637 0.60015,-0.93773 0.97525,-1.31282 0.39384,-0.35634 0.80645,-0.63766 1.27531,-0.82521 0.45012,-0.16879 0.93774,-0.26256 1.44411,-0.22505 0.52513,0.0187 1.03151,0.15003 1.50038,0.39384 0.31883,0.1688 0.60015,0.35634 0.84396,0.60015 0.24381,0.22506 0.43135,0.45012 0.58139,0.69393 l -0.71268,0.88147 -0.0375,-0.0188 c -0.0375,-0.0938 -0.0938,-0.2063 -0.15004,-0.33758 -0.0563,-0.13129 -0.13128,-0.26257 -0.24381,-0.39385 -0.0938,-0.13128 -0.2063,-0.26257 -0.33759,-0.39385 -0.15003,-0.13128 -0.31883,-0.26257 -0.52513,-0.35634 -0.63766,-0.33758 -1.29407,-0.31883 -1.96924,0 -0.6564
 1,0.33759 -1.2003,0.88147 -1.59415,1.65041 -0.22505,0.45011 -0.28132,0.86272 -0.16879,1.23781 0.0938,0.37509 0.3751,0.65641 0.80645,0.88147 0.2063,0.11253 0.43136,0.18755 0.67517,0.22506 0.22506,0.0563 0.43136,0.075 0.61891,0.075 0.18754,0.0188 0.37509,0.0188 0.56264,0 0.18754,-0.0188 0.31883,-0.0375 0.39384,-0.0375 l 0.0563,0.0187 -0.71268,0.91898 c -0.33758,-0.0188 -0.69392,-0.075 -1.06902,-0.13128 -0.35633,-0.075 -0.71267,-0.18755 -1.0315,-0.35634 z m 7.93322,3.88222 -1.12528,-0.56264 -0.52513,-3.30083 -1.10652,0.24381 -1.31283,1.55664 -0.91898,-0.46887 5.64516,-6.73292 0.91898,0.48762 -3.61966,4.31357 4.5949,-1.12528 1.21905,0.61891 -4.44485,0.99399 z m 8.30832,-0.46887 c -0.26256,0.54388 -0.6189,1.01275 -1.0315,1.4066 -0.39385,0.39385 -0.82521,0.67517 -1.27532,0.86271 -0.45011,0.20631 -0.91898,0.30008 -1.38785,0.31883 -0.48762,0 -0.91898,-0.0938 -1.35033,-0.31883 -0.28132,-0.15003 -0.54389,-0.31883 -0.75019,-0.50637 -0.22506,-0.18755 -0.39385,-0.4126 -0.52513,-0.63766 l -1.7066
 8,2.0255 -0.91898,-0.46886 5.55139,-6.60165 0.91898,0.46887 -0.43136,0.50638 c 0.4126,-0.0563 0.7877,-0.075 1.16279,-0.0563 0.35634,0.0188 0.71268,0.11253 1.06902,0.28132 0.54388,0.28132 0.86271,0.65641 0.97524,1.12528 0.11253,0.46887 0.0188,0.994 -0.30008,1.59415 z m -1.05026,-0.31883 c 0.2063,-0.37509 0.26257,-0.73143 0.2063,-1.03151 -0.0563,-0.28132 -0.28132,-0.52513 -0.65641,-0.71268 -0.26257,-0.15003 -0.5814,-0.22505 -0.93773,-0.24381 -0.35634,0 -0.71268,0 -1.05027,0.0563 l -2.28807,2.71943 c 0.15004,0.22505 0.31883,0.4126 0.46887,0.56264 0.16879,0.16879 0.39385,0.30007 0.65641,0.45011 0.33759,0.16879 0.69393,0.24381 1.05027,0.22505 0.33758,-0.0375 0.67516,-0.13128 0.97524,-0.30007 0.33758,-0.18755 0.6189,-0.4126 0.88147,-0.71268 0.26256,-0.28132 0.50638,-0.6189 0.69392,-1.01275 z m 7.05176,3.63841 c -0.24381,0.48762 -0.56264,0.91897 -0.93773,1.29407 -0.3751,0.37509 -0.7877,0.65641 -1.23781,0.84396 -0.46887,0.2063 -0.91898,0.31883 -1.4066,0.31883 -0.46887,0 -0.93774,-0.13129 -1
 .44411,-0.3751 -0.63766,-0.33758 -1.05027,-0.76894 -1.23781,-1.33158 -0.16879,-0.56264 -0.0938,-1.16279 0.24381,-1.8192 0.24381,-0.48762 0.56264,-0.91898 0.93773,-1.27532 0.3751,-0.37509 0.7877,-0.65641 1.23781,-0.86271 0.45011,-0.18755 0.91898,-0.30008 1.4066,-0.28132 0.48762,0 0.97524,0.11252 1.44411,0.35634 0.63766,0.31882 1.03151,0.75018 1.21905,1.29407 0.20631,0.56264 0.11253,1.16279 -0.22505,1.83796 z m -2.55064,1.29407 c 0.31883,-0.16879 0.60015,-0.39385 0.86272,-0.69392 0.28132,-0.30008 0.50637,-0.63766 0.69392,-1.01276 0.24381,-0.45011 0.30008,-0.86271 0.2063,-1.2003 -0.0938,-0.35633 -0.35634,-0.6189 -0.75018,-0.8252 -0.31883,-0.16879 -0.63766,-0.24381 -0.95649,-0.2063 -0.33759,0.0188 -0.65642,0.11252 -0.97525,0.28132 -0.30007,0.16879 -0.60015,0.4126 -0.86271,0.71267 -0.28132,0.28132 -0.50638,0.61891 -0.69392,0.994 -0.22506,0.45011 -0.30008,0.86272 -0.2063,1.2003 0.0938,0.35634 0.33758,0.63766 0.75018,0.84396 0.31883,0.15004 0.63766,0.22506 0.95649,0.2063 0.33758,-0.0188 0.
 65641,-0.11253 0.97524,-0.30007 z m 7.95198,-3.33833 -0.69392,0.84396 -1.05026,-0.52513 0.71268,-0.84396 z m -1.44411,1.6129 -4.051,4.8387 -0.91898,-0.46886 4.05101,-4.83871 z m 5.45761,4.36984 c -0.0375,0.075 -0.11252,0.16879 -0.18754,0.30007 -0.075,0.13128 -0.16879,0.22506 -0.24381,0.31883 l -2.62566,3.13203 -0.91898,-0.46887 2.30683,-2.73818 c 0.13128,-0.16879 0.22505,-0.30007 0.31883,-0.4126 0.0938,-0.11253 0.16879,-0.24381 0.22505,-0.37509 0.15004,-0.26257 0.18755,-0.50638 0.11253,-0.71268 -0.0563,-0.2063 -0.28132,-0.4126 -0.6189,-0.5814 -0.26257,-0.13128 -0.56264,-0.2063 -0.93774,-0.22505 -0.35633,-0.0188 -0.73143,0 -1.12528,0.0563 l -3.0195,3.6009 -0.91898,-0.46887 4.05101,-4.8387 0.91898,0.46886 -0.45011,0.54389 c 0.46887,-0.075 0.90022,-0.0938 1.27532,-0.0563 0.35634,0.0188 0.71267,0.11253 1.0315,0.28132 0.48762,0.24382 0.7877,0.54389 0.93774,0.91898 0.13128,0.3751 0.0938,0.7877 -0.13129,1.25657 z m 5.53263,1.23781 -0.54388,0.67516 -1.89422,-0.97524 -1.87547,2.23181 c -0.09
 38,0.0938 -0.2063,0.22505 -0.30008,0.39384 -0.11252,0.15004 -0.2063,0.26257 -0.24381,0.3751 -0.13128,0.22505 -0.15004,0.43136 -0.0938,0.6189 0.0563,0.16879 0.26256,0.33759 0.58139,0.50638 0.13129,0.075 0.30008,0.13128 0.50638,0.18755 0.2063,0.0375 0.35634,0.075 0.43136,0.075 l 0.0563,0.0188 -0.58139,0.71268 c -0.2063,-0.0563 -0.41261,-0.11253 -0.63766,-0.18755 -0.22506,-0.075 -0.41261,-0.15004 -0.5814,-0.24381 -0.45011,-0.22506 -0.75018,-0.50638 -0.88147,-0.82521 -0.15003,-0.31883 -0.11252,-0.69392 0.0938,-1.12528 0.0563,-0.11253 0.11252,-0.2063 0.18754,-0.30007 0.075,-0.0938 0.15004,-0.2063 0.24381,-0.31883 l 2.17555,-2.58815 -0.61891,-0.31883 0.54389,-0.65641 0.6189,0.31883 1.16279,-1.38785 0.93773,0.46887 -1.18154,1.38785 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3182"
+         d="m 229.5947,278.93824 -0.90023,-0.6189 1.06902,-2.04426 -2.88822,-2.00675 -2.4006,1.10652 -0.95649,-0.63766 8.66466,-3.95723 1.18155,0.80645 z m 0.60015,-3.46961 1.85671,-3.6009 -4.23856,1.95048 z m 2.36309,5.68266 c -0.31883,-0.22505 -0.5814,-0.45011 -0.76895,-0.69392 -0.2063,-0.24381 -0.33758,-0.50638 -0.39384,-0.80645 -0.075,-0.28132 -0.075,-0.56264 0,-0.88147 0.0563,-0.30008 0.2063,-0.60015 0.43135,-0.91898 0.31883,-0.48762 0.69393,-0.86272 1.12528,-1.18154 0.43136,-0.30008 0.90023,-0.52514 1.38785,-0.63766 0.46887,-0.11253 0.95649,-0.13129 1.46286,-0.0375 0.50638,0.0938 0.97525,0.28132 1.42536,0.60015 0.30007,0.2063 0.54388,0.43135 0.75019,0.69392 0.2063,0.26256 0.37509,0.50638 0.48762,0.76894 l -0.82521,0.7877 -0.0375,-0.0375 c -0.0375,-0.0938 -0.0563,-0.2063 -0.0938,-0.33759 -0.0563,-0.15003 -0.11253,-0.28132 -0.18755,-0.43135 -0.075,-0.15004 -0.18754,-0.30008 -0.30007,-0.45012 -0.11253,-0.15003 -0.28132,-0.28132 -0.46887,-0.4126 -0.58139,-0.4126 -1.23781,-0.48762 
 -1.95048,-0.26256 -0.69393,0.24381 -1.29408,0.71267 -1.7817,1.42535 -0.30007,0.4126 -0.4126,0.82521 -0.35634,1.2003 0.0563,0.37509 0.28132,0.71268 0.69393,0.994 0.18754,0.13128 0.39384,0.22505 0.6189,0.30007 0.22506,0.075 0.43136,0.13129 0.6189,0.16879 0.18755,0.0375 0.3751,0.0563 0.56264,0.0563 0.1688,0.0188 0.30008,0.0188 0.39385,0.0188 l 0.0375,0.0375 -0.80645,0.80645 c -0.35634,-0.075 -0.69392,-0.16879 -1.05026,-0.28132 -0.35634,-0.11252 -0.67517,-0.28132 -0.97524,-0.48762 z m 7.33307,4.91373 -1.0315,-0.71268 -0.075,-3.33833 -1.14404,0.0938 -1.50037,1.36909 -0.84396,-0.60015 6.48911,-5.88897 0.84397,0.5814 -4.16354,3.78844 4.70742,-0.46886 1.12528,0.76894 -4.53863,0.39385 z m 2.17554,1.51912 -1.08777,-0.75018 1.2003,-1.08777 1.08777,0.75018 z m 8.15829,5.81395 c -0.86272,-0.58139 -1.36909,-1.25656 -1.55664,-2.04426 -0.16879,-0.76894 0.0187,-1.57539 0.58139,-2.38184 0.43136,-0.6189 0.91898,-1.12528 1.48162,-1.53788 0.54389,-0.41261 1.14404,-0.69393 1.7817,-0.84396 0.6189,-0.1688 
 1.25656,-0.18755 1.93173,-0.0563 0.65641,0.11253 1.29407,0.39385 1.93173,0.82521 0.37509,0.26256 0.71268,0.56264 0.994,0.90022 0.28132,0.33759 0.52513,0.73143 0.75018,1.18155 l -0.90022,0.88147 -0.075,-0.0563 c -0.0188,-0.16879 -0.0563,-0.31883 -0.0938,-0.48762 -0.0563,-0.15004 -0.13128,-0.35634 -0.26257,-0.58139 -0.11253,-0.20631 -0.26256,-0.41261 -0.45011,-0.63766 -0.18755,-0.22506 -0.4126,-0.41261 -0.69392,-0.61891 -0.41261,-0.28132 -0.86272,-0.46886 -1.33158,-0.52513 -0.48763,-0.075 -0.95649,-0.0375 -1.42536,0.0938 -0.46887,0.13128 -0.93773,0.37509 -1.38785,0.71267 -0.45011,0.33759 -0.84396,0.75019 -1.18154,1.25657 -0.43136,0.6189 -0.60015,1.21905 -0.52513,1.78169 0.075,0.56264 0.43136,1.06902 1.03151,1.48162 0.30007,0.2063 0.58139,0.35634 0.88147,0.46887 0.30007,0.11252 0.60015,0.18754 0.88147,0.24381 0.26256,0.0375 0.50637,0.0563 0.73143,0.0563 0.22505,0.0188 0.43136,0.0188 0.60015,0 l 0.0563,0.0375 -0.91898,0.90022 c -0.43135,-0.0563 -0.90022,-0.16879 -1.38784,-0.31882 -0.506
 38,-0.1688 -0.994,-0.39385 -1.44411,-0.73144 z m 11.15903,1.80045 c -0.0563,0.075 -0.13129,0.16879 -0.22506,0.28132 -0.0938,0.11253 -0.18755,0.2063 -0.28132,0.28132 l -3.0195,2.75694 -0.86272,-0.5814 2.66317,-2.41935 c 0.15003,-0.13128 0.26256,-0.26256 0.37509,-0.35634 0.0938,-0.11253 0.18755,-0.22505 0.26257,-0.33758 0.18754,-0.26257 0.24381,-0.48762 0.2063,-0.71268 -0.0188,-0.2063 -0.2063,-0.43136 -0.54389,-0.65641 -0.22505,-0.15004 -0.52513,-0.26257 -0.88147,-0.33759 -0.35634,-0.0563 -0.73143,-0.0938 -1.12528,-0.0938 l -3.46961,3.15078 -0.84396,-0.58139 6.48911,-5.90772 0.84396,0.58139 -2.34433,2.13804 c 0.46887,0 0.90022,0.0375 1.25656,0.11252 0.3751,0.075 0.69393,0.22506 0.994,0.43136 0.45011,0.30008 0.71268,0.63766 0.80645,1.03151 0.0938,0.39385 -0.0188,0.80645 -0.30007,1.21905 z m 4.25731,4.16354 c 0.075,-0.075 0.13128,-0.13128 0.18754,-0.18755 0.0375,-0.0563 0.0938,-0.13128 0.15004,-0.2063 0.2063,-0.31883 0.30008,-0.63766 0.24381,-0.93773 -0.0375,-0.30008 -0.26256,-0.5814 -0
 .65641,-0.84396 -0.43136,-0.30008 -0.90023,-0.43136 -1.4066,-0.39385 -0.52513,0.0375 -0.994,0.22505 -1.4066,0.58139 z m -3.90097,2.0255 c -0.71268,-0.48762 -1.14404,-1.0315 -1.31283,-1.63165 -0.16879,-0.61891 -0.0375,-1.21906 0.4126,-1.85672 0.63766,-0.91897 1.48162,-1.51912 2.51313,-1.78169 1.05026,-0.24381 1.98799,-0.075 2.85071,0.52513 0.58139,0.39385 0.91898,0.82521 1.03151,1.33158 0.0938,0.48762 -0.0188,0.994 -0.3751,1.51913 -0.075,0.075 -0.18755,0.22506 -0.33758,0.39385 -0.16879,0.18755 -0.3751,0.37509 -0.60015,0.60015 l -3.71343,-2.58815 c -0.0938,0.075 -0.15003,0.1688 -0.22505,0.24381 -0.0563,0.075 -0.11253,0.15004 -0.1688,0.22506 -0.30007,0.43136 -0.39384,0.84396 -0.30007,1.27532 0.075,0.4126 0.37509,0.78769 0.84396,1.12528 0.33758,0.22506 0.71268,0.39385 1.16279,0.50638 0.43136,0.11252 0.82521,0.16879 1.18154,0.16879 l 0.0563,0.0375 -0.82521,0.78769 c -0.18754,-0.0375 -0.35634,-0.0938 -0.50637,-0.13128 -0.15004,-0.0375 -0.33759,-0.0938 -0.56264,-0.18755 -0.20631,-0.075 -0.
 41261,-0.15003 -0.56264,-0.22505 -0.1688,-0.075 -0.35634,-0.18755 -0.56264,-0.33759 z m 5.34508,3.69467 c -0.31883,-0.22505 -0.5814,-0.45011 -0.76894,-0.69392 -0.18755,-0.24381 -0.31883,-0.52513 -0.39385,-0.80645 -0.075,-0.28132 -0.075,-0.58139 0,-0.88147 0.0563,-0.30007 0.2063,-0.6189 0.43136,-0.93773 0.31882,-0.46887 0.71267,-0.86272 1.12528,-1.16279 0.43135,-0.30008 0.90022,-0.52513 1.38784,-0.63766 0.46887,-0.13128 0.95649,-0.13128 1.46287,-0.0563 0.50637,0.0938 0.99399,0.30008 1.42535,0.60015 0.30008,0.20631 0.54389,0.45012 0.75019,0.71268 0.2063,0.26257 0.37509,0.50638 0.48762,0.76894 l -0.80645,0.7877 -0.0563,-0.0375 c -0.0188,-0.0938 -0.0563,-0.22506 -0.0938,-0.35634 -0.0375,-0.13128 -0.11253,-0.26256 -0.18755,-0.4126 -0.075,-0.15004 -0.16879,-0.30008 -0.30007,-0.45011 -0.11253,-0.15004 -0.26257,-0.30008 -0.46887,-0.43136 -0.58139,-0.39385 -1.21905,-0.48762 -1.93173,-0.24381 -0.71268,0.24381 -1.31283,0.71268 -1.80045,1.4066 -0.30007,0.43136 -0.4126,0.8252 -0.35634,1.21905 0.
 0563,0.3751 0.30008,0.69392 0.69393,0.97524 0.18754,0.13129 0.39385,0.24382 0.63766,0.31883 0.22505,0.075 0.4126,0.13129 0.60015,0.1688 0.18754,0.0188 0.37509,0.0375 0.56264,0.0563 0.16879,0.0188 0.30007,0.0188 0.39384,0.0188 l 0.0563,0.0375 -0.82521,0.7877 c -0.35634,-0.0563 -0.69392,-0.15004 -1.05026,-0.26256 -0.35634,-0.11253 -0.67517,-0.28132 -0.97524,-0.48763 z m 7.33307,4.91373 -1.0315,-0.71268 -0.075,-3.33833 -1.14404,0.0938 -1.50037,1.36909 -0.84396,-0.60015 6.48912,-5.90772 0.86271,0.60015 -4.18229,3.78845 4.70742,-0.48763 1.12528,0.7877 -4.53863,0.37509 z m 8.30832,0.65641 c -0.33758,0.48762 -0.75018,0.91898 -1.20029,1.23781 -0.46887,0.33758 -0.93774,0.56264 -1.40661,0.69392 -0.46886,0.15004 -0.93773,0.18755 -1.4066,0.13129 -0.46886,-0.0563 -0.90022,-0.22506 -1.29407,-0.48763 -0.26256,-0.18754 -0.48762,-0.39384 -0.67517,-0.6189 -0.18754,-0.2063 -0.33758,-0.45011 -0.43135,-0.69392 l -1.96924,1.78169 -0.84396,-0.58139 6.37658,-5.81395 0.84396,0.60015 -0.48762,0.43135 c 0.412
 61,0 0.80645,0.0375 1.16279,0.0938 0.35634,0.075 0.69393,0.2063 1.03151,0.45011 0.48762,0.33758 0.75019,0.75019 0.80645,1.21905 0.0375,0.48762 -0.13128,0.994 -0.50638,1.55664 z m -0.99399,-0.45011 c 0.24381,-0.37509 0.35634,-0.69392 0.33758,-0.994 -0.0188,-0.30007 -0.2063,-0.58139 -0.54388,-0.8252 -0.26257,-0.1688 -0.56264,-0.28132 -0.91898,-0.33759 -0.33759,-0.075 -0.69393,-0.0938 -1.03151,-0.11253 l -2.64441,2.4006 c 0.13128,0.24381 0.26257,0.45011 0.39385,0.63766 0.15004,0.16879 0.33758,0.33758 0.60015,0.52513 0.30007,0.2063 0.63766,0.31883 0.994,0.33759 0.35633,0.0375 0.69392,-0.0188 1.01275,-0.15004 0.35634,-0.13128 0.67517,-0.33759 0.97524,-0.5814 0.30008,-0.24381 0.5814,-0.56264 0.82521,-0.90022 z m 6.48911,4.53863 c -0.30007,0.45011 -0.67516,0.82521 -1.10652,1.14403 -0.43136,0.33759 -0.86272,0.56265 -1.33158,0.69393 -0.48763,0.13128 -0.95649,0.16879 -1.42536,0.11253 -0.46887,-0.0563 -0.93773,-0.24381 -1.38784,-0.56264 -0.60015,-0.41261 -0.95649,-0.91898 -1.05027,-1.48162 -0.
 0938,-0.5814 0.0563,-1.16279 0.46887,-1.76294 0.31883,-0.45011 0.69392,-0.84396 1.10653,-1.16279 0.43135,-0.30008 0.88147,-0.54389 1.35033,-0.67517 0.46887,-0.13128 0.93774,-0.16879 1.42536,-0.0938 0.50637,0.0563 0.95649,0.24381 1.38784,0.54388 0.5814,0.39385 0.93774,0.88147 1.05026,1.46287 0.11253,0.56264 -0.0563,1.16279 -0.48762,1.78169 z m -2.70067,0.93773 c 0.33758,-0.11252 0.65641,-0.31883 0.95649,-0.56264 0.30007,-0.26256 0.58139,-0.56264 0.8252,-0.91898 0.28132,-0.43135 0.41261,-0.80645 0.35634,-1.16279 -0.0563,-0.35633 -0.26256,-0.67516 -0.6189,-0.91897 -0.30008,-0.20631 -0.61891,-0.31883 -0.93773,-0.33759 -0.33759,-0.0188 -0.65642,0.0375 -0.994,0.15004 -0.33759,0.13128 -0.65642,0.31883 -0.95649,0.58139 -0.31883,0.26257 -0.5814,0.56264 -0.82521,0.90023 -0.28132,0.4126 -0.4126,0.80645 -0.35633,1.18154 0.0375,0.35634 0.26256,0.65642 0.6189,0.91898 0.30007,0.2063 0.60015,0.31883 0.93773,0.33759 0.31883,0.0188 0.65642,-0.0375 0.994,-0.1688 z m 8.34583,-2.2318 -0.82521,0.73143 -0
 .95648,-0.65641 0.80645,-0.75019 z m -1.66917,1.4066 -4.65115,4.23855 -0.86272,-0.60015 4.66991,-4.23855 z m 4.81995,5.045 c -0.0375,0.075 -0.11252,0.1688 -0.22505,0.28132 -0.0938,0.11253 -0.18755,0.20631 -0.28132,0.28132 l -3.0195,2.75694 -0.84396,-0.60015 2.6444,-2.4006 c 0.15004,-0.15003 0.28132,-0.26256 0.3751,-0.37509 0.11253,-0.0938 0.18754,-0.2063 0.28132,-0.31883 0.16879,-0.26256 0.24381,-0.48762 0.2063,-0.71268 -0.0375,-0.2063 -0.22506,-0.43135 -0.54389,-0.65641 -0.22505,-0.15004 -0.52513,-0.26257 -0.88147,-0.33758 -0.37509,-0.0563 -0.75018,-0.0938 -1.12528,-0.11253 l -3.48836,3.16954 -0.84397,-0.5814 4.66992,-4.23855 0.84396,0.58139 -0.52513,0.46887 c 0.48762,0 0.90022,0.0375 1.27532,0.11253 0.35633,0.075 0.69392,0.2063 0.99399,0.4126 0.43136,0.31883 0.69393,0.65641 0.7877,1.05026 0.0938,0.39385 0,0.80645 -0.30008,1.21905 z m 5.32633,1.988 -0.6189,0.58139 -1.76294,-1.21905 -2.15679,1.95049 c -0.11253,0.0938 -0.22506,0.2063 -0.35634,0.35634 -0.13128,0.13128 -0.22505,0.24381
  -0.28132,0.33758 -0.15004,0.2063 -0.2063,0.4126 -0.16879,0.60015 0.0375,0.18755 0.2063,0.37509 0.50638,0.58139 0.11252,0.0938 0.28132,0.1688 0.46886,0.24382 0.2063,0.075 0.33759,0.13128 0.41261,0.15003 l 0.0563,0.0188 -0.65641,0.6189 c -0.20631,-0.075 -0.41261,-0.15004 -0.61891,-0.26256 -0.2063,-0.11253 -0.39385,-0.20631 -0.54388,-0.31883 -0.41261,-0.28132 -0.67517,-0.60015 -0.76895,-0.93774 -0.0938,-0.33758 -0.0187,-0.69392 0.26257,-1.10652 0.0563,-0.0938 0.13128,-0.16879 0.2063,-0.26257 0.0938,-0.0938 0.18755,-0.18755 0.28132,-0.28132 l 2.51313,-2.26931 -0.5814,-0.41261 0.63766,-0.58139 0.5814,0.4126 1.33158,-1.21905 0.86271,0.58139 -1.35033,1.21906 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3184"
+         d="m 410.35223,475.95607 c 0,-3.86346 14.96623,-6.99549 33.43958,-6.99549 18.47335,0 33.43958,3.13203 33.43958,6.99549 l 0,28.03824 c 0,3.86346 -14.96623,6.99549 -33.43958,6.99549 -18.47335,0 -33.43958,-3.13203 -33.43958,-6.99549 z"
+         style="fill:#d9d9d9;fill-opacity:1;fill-rule:evenodd;stroke:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3186"
+         d="m 477.23139,475.95607 c 0,3.88222 -14.96623,7.01425 -33.43958,7.01425 -18.47335,0 -33.43958,-3.13203 -33.43958,-7.01425"
+         style="fill:none;stroke:#000000;stroke-width:1.25656307px;stroke-linecap:butt;stroke-linejoin:miter;stroke-miterlimit:4;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3188"
+         d="m 410.35223,475.95607 c 0,-3.86346 14.96623,-6.99549 33.43958,-6.99549 18.47335,0 33.43958,3.13203 33.43958,6.99549 l 0,28.03824 c 0,3.86346 -14.96623,6.99549 -33.43958,6.99549 -18.47335,0 -33.43958,-3.13203 -33.43958,-6.99549 z"
+         style="fill:none;stroke:#000000;stroke-width:1.25656307px;stroke-linecap:butt;stroke-linejoin:miter;stroke-miterlimit:4;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <text
+         id="text3190"
+         style="font-size:11.2528038px;font-style:italic;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="526.71808"
+         x="402.59354"
+         xml:space="preserve">(snapshot store)</text>
+      <path
+         id="path3192"
+         d="m 358.25175,418.11666 -1.4066,8.73968 0,0 -0.67517,4.33233 0,-0.0188 -0.60015,4.27606 0,0 -0.54388,4.20105 0,0 -0.45012,4.10727 0,-0.0188 -0.33758,4.0135 0,-0.0188 -0.22506,3.88222 0,-0.0188 -0.075,3.73218 0,-0.0188 0.075,3.58214 0,-0.0375 0.26257,3.3946 -0.0188,-0.0375 0.46887,3.2258 0,-0.0375 0.30008,1.51913 -0.0188,-0.0375 0.37509,1.48162 -0.0187,-0.0375 0.43136,1.42536 -0.0188,-0.0375 0.48762,1.35033 -0.0188,-0.0188 0.54389,1.27532 -0.0188,-0.0375 0.60015,1.23781 -0.0187,-0.0375 0.67516,1.16279 -0.0187,-0.0375 0.75019,1.08777 -0.0375,-0.0375 0.8252,1.01275 -0.0375,-0.0375 0.88147,0.95649 -0.0375,-0.0375 0.95649,0.88147 -0.0375,-0.0188 1.01275,0.82521 -0.0375,-0.0375 1.06902,0.76894 -0.0375,-0.0187 1.14404,0.69392 -0.0375,-0.0188 1.2003,0.63765 -0.0375,-0.0188 1.25656,0.5814 -0.0375,-0.0188 2.64441,1.01275 -0.0375,-0.0188 2.83196,0.80645 -0.0375,0 3.0195,0.63766 -0.0375,-0.0188 3.18829,0.48762 -0.0188,0 3.31958,0.31883 -0.0188,0 3.46962,0.22506 -0.0375,0 3.56338,0.093
 8 -0.0188,0 3.65716,0.0188 -0.0187,0 3.71342,-0.0563 0,0 3.78844,-0.11252 -0.0188,0 3.88222,-0.15004 0.0563,1.23781 -3.88222,0.16879 -3.78844,0.0938 -3.73218,0.0563 -3.67592,0 -3.56338,-0.11253 -3.48837,-0.2063 -3.35709,-0.33759 -3.20705,-0.48762 -3.07576,-0.63766 -2.88822,-0.8252 -2.70068,-1.03151 -1.29407,-0.60015 -1.21905,-0.65641 -1.16279,-0.71268 -1.10653,-0.7877 -1.05026,-0.84396 -0.97524,-0.91898 -0.91898,-0.97524 -0.84396,-1.06902 -0.76894,-1.12528 -0.69393,-1.20029 -0.6189,-1.25657 -0.56264,-1.33158 -0.48762,-1.36909 -0.43136,-1.44411 -0.37509,-1.50037 -0.30008,-1.55664 -0.46886,-3.24456 -0.26257,-3.45086 -0.075,-3.61965 0.075,-3.75094 0.22506,-3.90097 0.35634,-4.03225 0.45011,-4.12603 0.54388,-4.2198 0.60015,-4.27607 0.67517,-4.33233 1.42536,-8.75843 1.21905,0.2063 z m 44.03597,65.11623 5.08252,2.28807 -4.89497,2.70067 -0.18755,-4.98874 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3194"
+         d="m 247.59918,420.55477 -0.37509,4.87622 0,-0.0188 -0.26257,4.85746 0,-0.0188 -0.0938,4.8387 0,-0.0187 0.0938,4.80119 0,-0.0187 0.13129,2.38184 0,-0.0187 0.18754,2.36309 0,-0.0188 0.24381,2.34434 0,-0.0188 0.30008,2.30683 0,-0.0188 0.37509,2.26932 0,-0.0188 0.45011,2.25056 0,-0.0188 0.52514,2.21305 0,-0.0188 0.6189,2.1943 -0.0187,-0.0375 0.71267,2.13803 -0.0188,-0.0187 0.78769,2.08177 -0.0187,-0.0188 0.88147,2.04426 -0.0188,-0.0188 0.97525,1.98799 -0.0188,-0.0188 1.08777,1.95048 -0.0188,-0.0375 1.16279,1.89423 -0.0188,-0.0188 1.29408,1.81921 -0.0188,-0.0188 1.38785,1.76294 -0.0188,-0.0188 1.50038,1.70667 -0.0188,-0.0188 1.63166,1.65041 -0.0375,-0.0375 1.76294,1.57539 -0.0375,-0.0188 1.87547,1.50037 -0.0188,-0.0188 1.988,1.42536 -0.0188,-0.0188 2.13803,1.33159 -0.0188,-0.0188 2.26932,1.27532 -0.0375,-0.0188 2.41935,1.18155 -0.0188,-0.0188 2.53188,1.10653 -0.0187,-0.0188 2.64441,1.01275 -0.0188,-0.0188 2.7757,0.93774 -0.0375,0 2.86946,0.84396 -0.0188,0 2.98199,0.76894 -0.018
 8,-0.0188 3.09453,0.71268 -0.0188,0 3.1883,0.61891 0,0 3.30082,0.56264 -0.0188,0 3.3946,0.48762 0,0 3.50712,0.45011 -0.0188,0 3.61965,0.37509 -0.0188,0 3.73218,0.31883 -0.0187,0 3.82595,0.28132 0,0 3.91973,0.22506 0,0 4.03225,0.16879 0,0 4.12603,0.15004 0,0 4.23856,0.0938 -0.0188,0 4.35109,0.0563 0,0 4.4261,0.0188 0,0 4.53863,-0.0188 0,0 4.6324,-0.0375 0,0 4.72618,-0.0563 0,0 4.83871,-0.0938 0,0 4.95123,-0.0938 0,0 5.02625,-0.13129 5.13878,-0.13128 5.2138,-0.15004 5.32633,-0.15003 5.4201,-0.15004 5.51387,-0.16879 5.60765,-0.1688 1.96924,-0.0563 0.0188,1.25656 -1.95049,0.0563 -5.60765,0.15003 -5.51387,0.1688 -5.4201,0.16879 -5.32633,0.15004 -5.23255,0.15003 -5.12003,0.13129 -5.045,0.11252 -4.93248,0.11253 -4.85746,0.075 -4.72618,0.075 -4.65116,0.0375 -4.53863,0 -4.44486,-0.0188 -4.35108,-0.0563 -4.23856,-0.0938 -4.14478,-0.13129 -4.03225,-0.18754 -3.93849,-0.22506 -3.8447,-0.26256 -3.73218,-0.31883 -3.61965,-0.39385 -3.52588,-0.45011 -3.41335,-0.48763 -3.31958,-0.56264 -3.20705,-0.63
 765 -3.11328,-0.71268 -3.00074,-0.76894 -2.88822,-0.86272 -2.79445,-0.93773 -2.68192,-1.01275 -2.55063,-1.10653 -2.45686,-1.18154 -2.28807,-1.29408 -2.1943,-1.35033 -2.02551,-1.44411 -1.89422,-1.53789 -1.78169,-1.59414 -1.65041,-1.66917 -1.53789,-1.74418 -1.4066,-1.80045 -1.31282,-1.85671 -1.18155,-1.91298 -1.10652,-1.988 -0.97525,-2.0255 -0.88147,-2.08177 -0.78769,-2.11928 -0.71268,-2.15679 -0.6189,-2.21305 -0.54389,-2.2318 -0.45011,-2.26932 -0.39385,-2.30682 -0.31883,-2.32558 -0.24381,-2.34434 -0.18755,-2.38184 -0.11252,-2.4006 -0.0938,-4.8387 0.0938,-4.85746 0.24381,-4.87622 0.39384,-4.87621 1.23781,0.0938 z m 153.61952,70.33002 5.06377,2.36309 -4.93248,2.64441 -0.13129,-5.0075 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <path
+         id="path3196"
+         d="m 104.81986,419.2607 0.0188,3.0195 0,-0.0375 0.15004,2.98199 0,-0.0375 0.30007,2.94449 -0.0187,-0.0375 0.46886,2.90697 0,-0.0188 0.60015,2.85071 -0.0187,-0.0375 0.75018,2.83195 0,-0.0375 0.90023,2.77569 0,-0.0188 1.05026,2.73818 -0.0187,-0.0375 1.20029,2.68191 -0.0187,-0.0188 1.35034,2.64441 -0.0188,-0.0375 1.50038,2.6069 -0.0188,-0.0375 1.65041,2.55063 -0.0188,-0.0187 1.8192,2.49437 -0.0187,-0.0187 1.95048,2.4381 -0.0187,-0.0187 2.10052,2.38184 -0.0187,-0.0187 2.25056,2.34433 -0.0188,-0.0188 2.4006,2.28807 -0.0188,-0.0188 2.55064,2.21305 -0.0187,0 2.70067,2.17555 -0.0188,-0.0188 2.86947,2.10052 -0.0188,-0.0188 3.00075,2.04426 -0.0188,0 3.15078,1.98799 -0.0188,-0.0188 3.31957,1.93173 -0.0188,-0.0188 3.45086,1.85671 -0.0188,0 3.61966,1.78169 -0.0188,0 3.75094,1.72543 -0.0188,-0.0188 3.91973,1.66916 -0.0188,-0.0188 4.06977,1.59415 -0.0188,0 4.2198,1.51912 -0.0187,-0.0188 4.36984,1.44411 -0.0188,0 4.51988,1.36909 -0.0188,0 4.66992,1.29407 -0.0188,0 9.62115,2.28807 -0.0375,0
  9.92122,1.89422 -0.0375,-0.0187 10.27756,1.51913 -0.0187,0 10.70892,1.2003 -0.0188,0 11.19654,0.88147 -0.0188,0 5.8327,0.33758 0,0 5.96398,0.28132 0,0 6.13278,0.22506 -0.0187,0 6.32032,0.18754 0,0 6.48912,0.11253 -0.0188,0 6.67667,0.0938 0,0 6.88296,0.0375 7.08927,0.0188 -0.0188,0 7.31433,-0.0188 7.52062,-0.0375 0,0 7.76443,-0.0563 8.00825,-0.075 8.27081,-0.0938 8.51462,-0.0938 8.79594,-0.0938 9.07726,-0.0938 9.37734,-0.075 9.65866,-0.075 9.97748,-0.0563 10.27756,-0.0375 10.61515,0 7.20179,0 0,1.25656 -7.20179,-0.0188 -10.59639,0.0188 -10.29632,0.0375 -9.97748,0.0563 -9.65866,0.0563 -9.35858,0.0938 -9.07726,0.0938 -8.79594,0.0938 -8.53338,0.0938 -8.25206,0.075 -8.00824,0.075 -7.76444,0.075 -7.53937,0.0375 -7.31433,0.0188 -7.08926,-0.0188 -6.86421,-0.0375 -6.69542,-0.0938 -6.48912,-0.13128 -6.32032,-0.16879 -6.13278,-0.22506 -5.98274,-0.28132 -5.8327,-0.35633 -11.2153,-0.88147 -10.72767,-1.18155 -10.29631,-1.53788 -9.93998,-1.89422 -9.6399,-2.30683 -4.68867,-1.29407 -4.53863,-1.3690
 9 -4.36984,-1.46287 -4.23856,-1.51912 -4.08851,-1.59415 -3.91973,-1.66917 -3.76969,-1.72543 -3.63841,-1.80045 -3.46961,-1.85671 -3.33833,-1.95048 -3.16954,-1.988 -3.0195,-2.06301 -2.88822,-2.11928 -2.71943,-2.1943 -2.56939,-2.2318 -2.43811,-2.30683 -2.26931,-2.36309 -2.11928,-2.41935 -1.96924,-2.45686 -1.81921,-2.53188 -1.66916,-2.56939 -1.51913,-2.62566 -1.36909,-2.68192 -1.2003,-2.71942 -1.06902,-2.75694 -0.91898,-2.8132 -0.75018,-2.85071 -0.61891,-2.90698 -0.46886,-2.92572 -0.30008,-2.982 -0.15004,-3.00074 0,-3.03826 1.23781,0 z m 298.19929,77.96317 4.98874,2.51313 -5.00749,2.49437 0.0188,-5.0075 z"
+         style="fill:#000000;fill-opacity:1;fill-rule:nonzero;stroke:#000000;stroke-width:0.01875467px;stroke-linecap:butt;stroke-linejoin:round;stroke-opacity:1;stroke-dasharray:none"
+         inkscape:connector-curvature="0" />
+      <text
+         id="text3198"
+         style="font-size:11.2528038px;font-style:italic;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="468.82831"
+         x="182.75459"
+         xml:space="preserve">store state</text>
+      <text
+         id="text3200"
+         style="font-size:11.2528038px;font-style:italic;font-weight:normal;text-align:start;text-anchor:start;fill:#000000;font-family:Verdana"
+         y="482.33167"
+         x="184.85512"
+         xml:space="preserve">snapshots</text>
+    </g>
+  </g>
+</svg>