You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@nifi.apache.org by mc...@apache.org on 2018/04/03 19:44:47 UTC

[1/5] nifi git commit: NIFI-4932: Enable S2S work behind a Reverse Proxy Adding S2S endpoint Reverse Proxy mapping capability. Added license header to SVG files. Incorporated review comments. Use regex to check property key processing. Catch AttributeExp

Repository: nifi
Updated Branches:
  refs/heads/master 7c0ee014d -> 1913b1e2a


http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/PeerDescriptionModifiable.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/PeerDescriptionModifiable.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/PeerDescriptionModifiable.java
new file mode 100644
index 0000000..ef9cac0
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/PeerDescriptionModifiable.java
@@ -0,0 +1,25 @@
+/*
+ * 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.
+ */
+package org.apache.nifi.remote;
+
+/**
+ * This interface is used to determine whether a ServerProtocol implementation
+ * can utilize peer description modification for making S2S work behind a reverse proxy.
+ */
+public interface PeerDescriptionModifiable {
+    void setPeerDescriptionModifier(final PeerDescriptionModifier modifier);
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/PeerDescriptionModifier.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/PeerDescriptionModifier.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/PeerDescriptionModifier.java
new file mode 100644
index 0000000..9e40c49
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/PeerDescriptionModifier.java
@@ -0,0 +1,182 @@
+/*
+ * 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.
+ */
+package org.apache.nifi.remote;
+
+import org.apache.nifi.attribute.expression.language.PreparedQuery;
+import org.apache.nifi.attribute.expression.language.Query;
+import org.apache.nifi.attribute.expression.language.exception.AttributeExpressionLanguageParsingException;
+import org.apache.nifi.remote.protocol.SiteToSiteTransportProtocol;
+import org.apache.nifi.util.NiFiProperties;
+import org.apache.nifi.util.Tuple;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.util.List;
+import java.util.Map;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+import java.util.stream.Collectors;
+
+import static java.lang.String.format;
+import static org.apache.commons.lang3.StringUtils.isBlank;
+
+public class PeerDescriptionModifier {
+
+    private static final Logger logger = LoggerFactory.getLogger(PeerDescriptionModifier.class);
+
+    public enum RequestType {
+        SiteToSiteDetail,
+        Peers
+    }
+
+    private static class Route {
+        private String name;
+        private SiteToSiteTransportProtocol protocol;
+        private PreparedQuery predicate;
+        private PreparedQuery hostname;
+        private PreparedQuery port;
+        private PreparedQuery secure;
+
+        private Route validate() {
+            if (hostname == null) {
+                throw new IllegalArgumentException(
+                        format("Found an invalid Site-to-Site route definition [%s] 'hostname' is not specified.", name));
+            }
+            if (port == null) {
+                throw new IllegalArgumentException(
+                        format("Found an invalid Site-to-Site route definition [%s] 'port' is not specified.", name));
+            }
+            return this;
+        }
+
+        private PeerDescription getTarget(final Map<String, String> variables) {
+            final String targetHostName = hostname.evaluateExpressions(variables, null);
+            if (isBlank(targetHostName)) {
+                throw new IllegalStateException("Target hostname was not resolved for the route definition " + name);
+            }
+
+            final String targetPortStr = port.evaluateExpressions(variables, null);
+            if (isBlank(targetPortStr)) {
+                throw new IllegalStateException("Target port was not resolved for the route definition " + name);
+            }
+
+            final String targetIsSecure = secure == null ? null : secure.evaluateExpressions(variables, null);
+            return new PeerDescription(targetHostName, Integer.valueOf(targetPortStr), Boolean.valueOf(targetIsSecure));
+        }
+    }
+
+    private Map<SiteToSiteTransportProtocol, List<Route>> routes;
+
+
+    private static final String PROPERTY_PREFIX = "nifi.remote.route.";
+    private static final Pattern PROPERTY_REGEX = Pattern.compile("^nifi\\.remote\\.route\\.(raw|http)\\.([^.]+)\\.(when|hostname|port|secure)$");
+
+    public PeerDescriptionModifier(final NiFiProperties properties) {
+        final Map<Tuple<String, String>, List<Tuple<String, String>>> routeDefinitions = properties.getPropertyKeys().stream()
+                .filter(propertyKey -> propertyKey.startsWith(PROPERTY_PREFIX))
+                .map(propertyKey -> {
+                            final Matcher matcher = PROPERTY_REGEX.matcher(propertyKey);
+                            if (!matcher.matches()) {
+                                throw new IllegalArgumentException(
+                                        format("Found an invalid Site-to-Site route definition property '%s'." +
+                                                        " Routing property keys should be formatted as 'nifi.remote.route.{protocol}.{name}.{routingConfigName}'." +
+                                                        " Where {protocol} is 'raw' or 'http', and {routingConfigName} is 'when', 'hostname', 'port' or 'secure'.",
+                                                propertyKey));
+                            }
+                            return matcher;
+                })
+                .collect(Collectors.groupingBy(matcher -> new Tuple<>(matcher.group(1), matcher.group(2)),
+                        Collectors.mapping(matcher -> new Tuple<>(matcher.group(3), matcher.group(0)), Collectors.toList())));
+
+        routes = routeDefinitions.entrySet().stream().map(routeDefinition -> {
+            final Route route = new Route();
+            // E.g. [raw, example1], [http, example2]
+            final Tuple<String, String> protocolAndRoutingName = routeDefinition.getKey();
+            route.protocol = SiteToSiteTransportProtocol.valueOf(protocolAndRoutingName.getKey().toUpperCase());
+            route.name = protocolAndRoutingName.getValue();
+            routeDefinition.getValue().forEach(routingConfigNameAndPropertyKey -> {
+                final String routingConfigName = routingConfigNameAndPropertyKey.getKey();
+                final String propertyKey = routingConfigNameAndPropertyKey.getValue();
+                final String routingConfigValue = properties.getProperty(propertyKey);
+                try {
+                    switch (routingConfigName) {
+                        case "when":
+                            route.predicate = Query.prepare(routingConfigValue);
+                            break;
+                        case "hostname":
+                            route.hostname = Query.prepare(routingConfigValue);
+                            break;
+                        case "port":
+                            route.port = Query.prepare(routingConfigValue);
+                            break;
+                        case "secure":
+                            route.secure = Query.prepare(routingConfigValue);
+                            break;
+                    }
+                } catch (AttributeExpressionLanguageParsingException e) {
+                    throw new IllegalArgumentException(format("Failed to parse NiFi expression language configured" +
+                            " for Site-to-Site routing property at '%s' due to '%s'", propertyKey, e.getMessage()), e);
+                }
+            });
+            return route;
+        }).map(Route::validate).collect(Collectors.groupingBy(r -> r.protocol));
+
+    }
+
+    private void addVariables(Map<String, String> map, String prefix, PeerDescription peer) {
+        map.put(format("%s.hostname", prefix), peer.getHostname());
+        map.put(format("%s.port", prefix), String.valueOf(peer.getPort()));
+        map.put(format("%s.secure", prefix), String.valueOf(peer.isSecure()));
+    }
+
+    public boolean isModificationNeeded(final SiteToSiteTransportProtocol protocol) {
+        return routes != null && routes.containsKey(protocol) && !routes.get(protocol).isEmpty();
+    }
+
+    /**
+     * Modifies target peer description so that subsequent request can go through the appropriate route
+     * @param source The source peer from which a request was sent, this can be any server host participated to relay the request,
+     *              but should be the one which can contribute to derive the correct target peer.
+     * @param target The original target which should receive and process further incoming requests.
+     * @param protocol The S2S protocol being used.
+     * @param requestType The requested API type.
+     * @param variables Containing context variables those can be referred from Expression Language.
+     * @return A peer description. The original target peer can be returned if there is no intermediate peer such as reverse proxies needed.
+     */
+    public PeerDescription modify(final PeerDescription source, final PeerDescription target,
+                                  final SiteToSiteTransportProtocol protocol, final RequestType requestType,
+                                  final Map<String, String> variables) {
+
+        addVariables(variables, "s2s.source", source);
+        addVariables(variables, "s2s.target", target);
+        variables.put("s2s.protocol", protocol.name());
+        variables.put("s2s.request", requestType.name());
+
+        logger.debug("Modifying PeerDescription, variables={}", variables);
+
+        return routes.get(protocol).stream().filter(r -> r.predicate == null
+                || Boolean.valueOf(r.predicate.evaluateExpressions(variables, null)))
+                .map(r -> {
+                    final PeerDescription t = r.getTarget(variables);
+                    logger.debug("Route definition {} matched, {}", r.name, t);
+                    return t;
+                })
+                // If a matched route was found, use it, else use the original target.
+                .findFirst().orElse(target);
+
+    }
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/SocketRemoteSiteListener.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/SocketRemoteSiteListener.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/SocketRemoteSiteListener.java
index 2fae669..07c5920 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/SocketRemoteSiteListener.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/SocketRemoteSiteListener.java
@@ -64,6 +64,7 @@ public class SocketRemoteSiteListener implements RemoteSiteListener {
     private final NodeInformant nodeInformant;
     private final AtomicReference<ProcessGroup> rootGroup = new AtomicReference<>();
     private final NiFiProperties nifiProperties;
+    private final PeerDescriptionModifier peerDescriptionModifier;
 
     private final AtomicBoolean stopped = new AtomicBoolean(false);
 
@@ -78,6 +79,7 @@ public class SocketRemoteSiteListener implements RemoteSiteListener {
         this.sslContext = sslContext;
         this.nifiProperties = nifiProperties;
         this.nodeInformant = nodeInformant;
+        peerDescriptionModifier = new PeerDescriptionModifier(nifiProperties);
     }
 
     @Override
@@ -218,6 +220,9 @@ public class SocketRemoteSiteListener implements RemoteSiteListener {
                                 protocol = RemoteResourceFactory.receiveServerProtocolNegotiation(dis, dos);
                                 protocol.setRootProcessGroup(rootGroup.get());
                                 protocol.setNodeInformant(nodeInformant);
+                                if (protocol instanceof PeerDescriptionModifiable) {
+                                    ((PeerDescriptionModifiable)protocol).setPeerDescriptionModifier(peerDescriptionModifier);
+                                }
 
                                 final PeerDescription description = new PeerDescription(clientHostName, clientPort, sslContext != null);
                                 peer = new Peer(description, commsSession, peerUri, "nifi://localhost:" + getPort());

http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/protocol/socket/SocketFlowFileServerProtocol.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/protocol/socket/SocketFlowFileServerProtocol.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/protocol/socket/SocketFlowFileServerProtocol.java
index a7c0212..cb0746e 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/protocol/socket/SocketFlowFileServerProtocol.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/main/java/org/apache/nifi/remote/protocol/socket/SocketFlowFileServerProtocol.java
@@ -26,6 +26,9 @@ import java.util.List;
 import java.util.Map;
 import java.util.Optional;
 import org.apache.nifi.remote.Peer;
+import org.apache.nifi.remote.PeerDescription;
+import org.apache.nifi.remote.PeerDescriptionModifiable;
+import org.apache.nifi.remote.PeerDescriptionModifier;
 import org.apache.nifi.remote.RemoteResourceFactory;
 import org.apache.nifi.remote.StandardVersionNegotiator;
 import org.apache.nifi.remote.VersionNegotiator;
@@ -39,14 +42,22 @@ import org.apache.nifi.remote.protocol.CommunicationsSession;
 import org.apache.nifi.remote.protocol.HandshakeProperties;
 import org.apache.nifi.remote.protocol.RequestType;
 import org.apache.nifi.remote.protocol.ResponseCode;
+import org.apache.nifi.remote.protocol.SiteToSiteTransportProtocol;
 
-public class SocketFlowFileServerProtocol extends AbstractFlowFileServerProtocol {
+public class SocketFlowFileServerProtocol extends AbstractFlowFileServerProtocol implements PeerDescriptionModifiable {
 
     public static final String RESOURCE_NAME = "SocketFlowFileProtocol";
 
     // Version 6 added to support Zero-Master Clustering, which was introduced in NiFi 1.0.0
     private final VersionNegotiator versionNegotiator = new StandardVersionNegotiator(6, 5, 4, 3, 2, 1);
 
+    private PeerDescriptionModifier peerDescriptionModifier;
+
+    @Override
+    public void setPeerDescriptionModifier(PeerDescriptionModifier modifier) {
+        peerDescriptionModifier = modifier;
+    }
+
     @Override
     protected HandshakeProperties doHandshake(Peer peer) throws IOException, HandshakeException {
 
@@ -189,9 +200,21 @@ public class SocketFlowFileServerProtocol extends AbstractFlowFileServerProtocol
                 continue;
             }
 
-            dos.writeUTF(nodeInfo.getSiteToSiteHostname());
-            dos.writeInt(nodeInfo.getSiteToSitePort());
-            dos.writeBoolean(nodeInfo.isSiteToSiteSecure());
+            if (peerDescriptionModifier != null && peerDescriptionModifier.isModificationNeeded(SiteToSiteTransportProtocol.RAW)) {
+                final PeerDescription target = new PeerDescription(nodeInfo.getSiteToSiteHostname(), nodeInfo.getSiteToSitePort(), nodeInfo.isSiteToSiteSecure());
+                final PeerDescription modifiedTarget = peerDescriptionModifier.modify(peer.getDescription(), target,
+                        SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.Peers, new HashMap<>());
+
+                dos.writeUTF(modifiedTarget.getHostname());
+                dos.writeInt(modifiedTarget.getPort());
+                dos.writeBoolean(modifiedTarget.isSecure());
+
+            } else {
+                dos.writeUTF(nodeInfo.getSiteToSiteHostname());
+                dos.writeInt(nodeInfo.getSiteToSitePort());
+                dos.writeBoolean(nodeInfo.isSiteToSiteSecure());
+            }
+
             dos.writeInt(nodeInfo.getTotalFlowFiles());
         }
 

http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/test/java/org/apache/nifi/remote/TestPeerDescriptionModifier.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/test/java/org/apache/nifi/remote/TestPeerDescriptionModifier.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/test/java/org/apache/nifi/remote/TestPeerDescriptionModifier.java
new file mode 100644
index 0000000..cf6639f
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-site-to-site/src/test/java/org/apache/nifi/remote/TestPeerDescriptionModifier.java
@@ -0,0 +1,321 @@
+/*
+ * 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.
+ */
+package org.apache.nifi.remote;
+
+import org.apache.nifi.attribute.expression.language.exception.AttributeExpressionLanguageException;
+import org.apache.nifi.properties.StandardNiFiProperties;
+import org.apache.nifi.remote.protocol.SiteToSiteTransportProtocol;
+import org.apache.nifi.util.NiFiProperties;
+import org.junit.Test;
+
+import java.util.HashMap;
+import java.util.Map;
+import java.util.Properties;
+
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertFalse;
+import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertTrue;
+import static org.junit.Assert.fail;
+
+public class TestPeerDescriptionModifier {
+
+    @Test
+    public void testNoConfiguration() {
+        Properties props = new Properties();
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        final PeerDescriptionModifier modifier = new PeerDescriptionModifier(properties);
+        assertFalse(modifier.isModificationNeeded(SiteToSiteTransportProtocol.RAW));
+        assertFalse(modifier.isModificationNeeded(SiteToSiteTransportProtocol.HTTP));
+    }
+
+    @Test
+    public void testInvalidNoHostname() {
+        Properties props = new Properties();
+        props.put("nifi.remote.route.raw.no-host.when", "true");
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        try {
+            new PeerDescriptionModifier(properties);
+            fail("Should throw an Exception");
+        } catch (IllegalArgumentException e) {
+            assertEquals("Found an invalid Site-to-Site route definition [no-host] 'hostname' is not specified.", e.getMessage());
+        }
+    }
+
+    @Test
+    public void testInvalidNoPort() {
+        Properties props = new Properties();
+        props.put("nifi.remote.route.raw.no-port.when", "true");
+        props.put("nifi.remote.route.raw.no-port.hostname", "proxy.example.com");
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        try {
+            new PeerDescriptionModifier(properties);
+            fail("Should throw an Exception");
+        } catch (IllegalArgumentException e) {
+            assertEquals("Found an invalid Site-to-Site route definition [no-port] 'port' is not specified.", e.getMessage());
+        }
+    }
+
+    @Test
+    public void testInvalidConfigurationName() {
+        Properties props = new Properties();
+        props.put("nifi.remote.route.raw.invalid-name.when", "true");
+        props.put("nifi.remote.route.raw.invalid-name.hostname", "proxy.example.com");
+        props.put("nifi.remote.route.raw.invalid-name.port", "8081");
+        props.put("nifi.remote.route.raw.invalid-name.secure", "true");
+        props.put("nifi.remote.route.raw.invalid-name.unsupported", "true");
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        try {
+            new PeerDescriptionModifier(properties);
+            fail("Should throw an Exception");
+        } catch (IllegalArgumentException e) {
+            assertEquals("Found an invalid Site-to-Site route definition property 'nifi.remote.route.raw.invalid-name.unsupported'." +
+                    " Routing property keys should be formatted as 'nifi.remote.route.{protocol}.{name}.{routingConfigName}'." +
+                    " Where {protocol} is 'raw' or 'http', and {routingConfigName} is 'when', 'hostname', 'port' or 'secure'.", e.getMessage());
+        }
+    }
+
+    @Test
+    public void testInvalidPropertyKeyNoProtocol() {
+        Properties props = new Properties();
+        props.put("nifi.remote.route.", "true");
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        try {
+            new PeerDescriptionModifier(properties);
+            fail("Should throw an Exception");
+        } catch (IllegalArgumentException e) {
+            assertEquals("Found an invalid Site-to-Site route definition property 'nifi.remote.route.'." +
+                    " Routing property keys should be formatted as 'nifi.remote.route.{protocol}.{name}.{routingConfigName}'." +
+                    " Where {protocol} is 'raw' or 'http', and {routingConfigName} is 'when', 'hostname', 'port' or 'secure'.", e.getMessage());
+        }
+    }
+
+    @Test
+    public void testInvalidPropertyKeyNoName() {
+        Properties props = new Properties();
+        props.put("nifi.remote.route.http.", "true");
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        try {
+            new PeerDescriptionModifier(properties);
+            fail("Should throw an Exception");
+        } catch (IllegalArgumentException e) {
+            assertEquals("Found an invalid Site-to-Site route definition property 'nifi.remote.route.http.'." +
+                    " Routing property keys should be formatted as 'nifi.remote.route.{protocol}.{name}.{routingConfigName}'." +
+                    " Where {protocol} is 'raw' or 'http', and {routingConfigName} is 'when', 'hostname', 'port' or 'secure'.", e.getMessage());
+        }
+    }
+
+    @Test
+    public void testInvalidExpression() {
+        Properties props = new Properties();
+        props.put("nifi.remote.route.raw.invalid-el.when", "${nonExistingFunction()}");
+        props.put("nifi.remote.route.raw.invalid-el.hostname", "proxy.example.com");
+        props.put("nifi.remote.route.raw.invalid-el.port", "8081");
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        final PeerDescriptionModifier modifier = new PeerDescriptionModifier(properties);
+
+        final PeerDescription source = new PeerDescription("client", 12345, true);
+        final PeerDescription target = new PeerDescription("nifi0", 8081, true);
+
+        try {
+            modifier.modify(source, target,
+                    SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.Peers, new HashMap<>());
+            fail("Should throw an Exception");
+        } catch (AttributeExpressionLanguageException e) {
+            assertTrue(e.getMessage().startsWith("Invalid Expression"));
+        }
+    }
+
+    @Test
+    public void testDefaultIsNotSecure() {
+        Properties props = new Properties();
+        props.put("nifi.remote.route.raw.no-port.when", "true");
+        props.put("nifi.remote.route.raw.no-port.hostname", "proxy.example.com");
+        props.put("nifi.remote.route.raw.no-port.port", "8443");
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        final PeerDescriptionModifier modifier = new PeerDescriptionModifier(properties);
+
+        final PeerDescription source = new PeerDescription("client", 12345, true);
+        final PeerDescription target = new PeerDescription("nifi0", 8081, true);
+        final PeerDescription modifiedTarget = modifier.modify(source, target,
+                SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.Peers, new HashMap<>());
+        assertFalse(modifiedTarget.isSecure());
+    }
+
+    @Test
+    public void testRawPortToNode() {
+        Properties props = new Properties();
+
+        // RAW S2S route configs.
+        // Port number to Node
+        // proxy1.example.com:17491 -> nifi0:8081
+        // proxy1.example.com:17492 -> nifi1:8081
+        props.put("nifi.remote.route.raw.port-to-node.when", "${X-ProxyHost:equals('proxy1.example.com')" +
+                ":or(${s2s.source.hostname:equals('proxy1.example.com')})}");
+        props.put("nifi.remote.route.raw.port-to-node.hostname", "proxy1.example.com");
+        props.put("nifi.remote.route.raw.port-to-node.port",
+                "${s2s.target.hostname:equals('nifi0'):ifElse('17491'," +
+                        "${s2s.target.hostname:equals('nifi1'):ifElse('17492', 'undefined')})}");
+        props.put("nifi.remote.route.raw.port-to-node.secure", "true");
+
+        // Other S2S configs.
+        props.put("nifi.remote.input.host", "node0");
+        props.put("nifi.remote.input.secure", "true");
+        props.put("nifi.remote.input.socket.port", "8081");
+        props.put("nifi.remote.input.http.enabled", "true");
+
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        final PeerDescriptionModifier modifier = new PeerDescriptionModifier(properties);
+
+        // For requests coming from the proxy server, modify target description,
+        // so that client can send further request to the proxy.
+        // To nifi0.
+        PeerDescription source = new PeerDescription("proxy1.example.com", 12345, true);
+        PeerDescription target = new PeerDescription("nifi0", 8081, true);
+        PeerDescription modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>());
+
+        assertNotNull(modifiedTarget);
+        assertEquals("proxy1.example.com", modifiedTarget.getHostname());
+        assertEquals(17491, modifiedTarget.getPort());
+        assertEquals(true, modifiedTarget.isSecure());
+
+        // To nifi1.
+        target = new PeerDescription("nifi1", 8081, true);
+        modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>());
+
+        assertNotNull(modifiedTarget);
+        assertEquals("proxy1.example.com", modifiedTarget.getHostname());
+        assertEquals(17492, modifiedTarget.getPort());
+        assertEquals(true, modifiedTarget.isSecure());
+
+        // For requests coming directly, use the original target description.
+        source = new PeerDescription("192.168.1.101", 23456, true);
+        modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>());
+        assertNotNull(modifiedTarget);
+        assertEquals(target, modifiedTarget);
+
+    }
+
+    @Test
+    public void testRawServerNameToNode() {
+        Properties props = new Properties();
+
+        // RAW S2S route configs.
+        // Server name to Node
+        // nifi0.example.com:17491 -> nifi0:8081
+        // nifi1.example.com:17491 -> nifi1:8081
+        props.put("nifi.remote.route.raw.name-to-node.when", "${X-ProxyHost:contains('.example.com')" +
+                ":or(${s2s.source.hostname:contains('.example.com')})}");
+        props.put("nifi.remote.route.raw.name-to-node.hostname", "${s2s.target.hostname}.example.com");
+        props.put("nifi.remote.route.raw.name-to-node.port", "17491");
+        props.put("nifi.remote.route.raw.name-to-node.secure", "true");
+
+        // Other S2S configs.
+        props.put("nifi.remote.input.host", "node0");
+        props.put("nifi.remote.input.secure", "true");
+        props.put("nifi.remote.input.socket.port", "8081");
+        props.put("nifi.remote.input.http.enabled", "true");
+
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        final PeerDescriptionModifier modifier = new PeerDescriptionModifier(properties);
+
+        // For requests coming from the proxy server, modify target description,
+        // so that client can send further request to the proxy.
+        // To nifi0.
+        PeerDescription source = new PeerDescription("nifi0.example.com", 12345, true);
+        PeerDescription target = new PeerDescription("nifi0", 8081, true);
+        PeerDescription modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>());
+
+        assertNotNull(modifiedTarget);
+        assertEquals("nifi0.example.com", modifiedTarget.getHostname());
+        assertEquals(17491, modifiedTarget.getPort());
+        assertEquals(true, modifiedTarget.isSecure());
+
+        // To nifi1.
+        target = new PeerDescription("nifi1", 8081, true);
+        modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>());
+
+        assertNotNull(modifiedTarget);
+        assertEquals("nifi1.example.com", modifiedTarget.getHostname());
+        assertEquals(17491, modifiedTarget.getPort());
+        assertEquals(true, modifiedTarget.isSecure());
+
+        // For requests coming directly, use the original target description.
+        source = new PeerDescription("192.168.1.101", 23456, true);
+        modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>());
+        assertNotNull(modifiedTarget);
+        assertEquals(target, modifiedTarget);
+
+    }
+
+    @Test
+    public void testHttpsTerminate() {
+        Properties props = new Properties();
+
+        // https://nifi0.example.com -> http://nifi0:8080
+        // https://nifi1.example.com -> http://nifi1:8080
+        // S2S HTTP configs.
+        props.put("nifi.remote.route.http.terminate.when", "${X-ProxyHost:contains('.example.com')" +
+                ":or(${s2s.source.hostname:contains('.example.com')})}");
+        props.put("nifi.remote.route.http.terminate.hostname", "${s2s.target.hostname}.example.com");
+        props.put("nifi.remote.route.http.terminate.port", "443");
+        props.put("nifi.remote.route.http.terminate.secure", "true");
+
+        // Other S2S configs.
+        props.put("nifi.web.http.host", "nifi0");
+        props.put("nifi.web.http.port", "8080");
+        props.put("nifi.remote.input.host", "nifi0");
+        props.put("nifi.remote.input.secure", "false");
+        props.put("nifi.remote.input.socket.port", "");
+        props.put("nifi.remote.input.http.enabled", "true");
+
+
+        final NiFiProperties properties = new StandardNiFiProperties(props);
+        final PeerDescriptionModifier modifier = new PeerDescriptionModifier(properties);
+
+        // For requests coming from the proxy server, modify target description,
+        // so that client can send further request to the proxy.
+        // To nifi0.
+        PeerDescription source = new PeerDescription("nifi0.example.com", 12345, true);
+        PeerDescription target = new PeerDescription("nifi0", 8080, false);
+        final Map<String, String> proxyHeders = new HashMap<>();
+        proxyHeders.put("X-ProxyHost", "nifi0.example.com:443");
+        proxyHeders.put("X-Forwarded-For", "172.16.1.103");
+        PeerDescription modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.HTTP, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>(proxyHeders));
+
+        assertNotNull(modifiedTarget);
+        assertEquals("nifi0.example.com", modifiedTarget.getHostname());
+        assertEquals(443, modifiedTarget.getPort());
+        assertEquals(true, modifiedTarget.isSecure());
+
+        // To nifi1.
+        proxyHeders.put("X-ProxyHost", "nifi1.example.com:443");
+        target = new PeerDescription("nifi1", 8081, true);
+        modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.HTTP, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>(proxyHeders));
+
+        assertNotNull(modifiedTarget);
+        assertEquals("nifi1.example.com", modifiedTarget.getHostname());
+        assertEquals(443, modifiedTarget.getPort());
+        assertEquals(true, modifiedTarget.isSecure());
+
+        // For requests coming directly, use the original target description.
+        source = new PeerDescription("192.168.1.101", 23456, true);
+        modifiedTarget = modifier.modify(source, target, SiteToSiteTransportProtocol.HTTP, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>());
+        assertNotNull(modifiedTarget);
+        assertEquals(target, modifiedTarget);
+    }
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ApplicationResource.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ApplicationResource.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ApplicationResource.java
index db0a568..85423c0 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ApplicationResource.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ApplicationResource.java
@@ -137,6 +137,11 @@ public abstract class ApplicationResource {
      * @return resource uri
      */
     protected String generateResourceUri(final String... path) {
+        URI uri = buildResourceUri(path);
+        return uri.toString();
+    }
+
+    private URI buildResourceUri(final String... path) {
         final UriBuilder uriBuilder = uriInfo.getBaseUriBuilder();
         uriBuilder.segment(path);
         URI uri = uriBuilder.build();
@@ -179,7 +184,7 @@ public abstract class ApplicationResource {
         } catch (final URISyntaxException use) {
             throw new UriBuilderException(use);
         }
-        return uri.toString();
+        return uri;
     }
 
     /**
@@ -1226,9 +1231,9 @@ public abstract class ApplicationResource {
         public Response locationResponse(UriInfo uriInfo, String portType, String portId, String transactionId, Object entity,
                                          Integer protocolVersion, final HttpRemoteSiteListener transactionManager) {
 
-            String path = "/data-transfer/" + portType + "/" + portId + "/transactions/" + transactionId;
-            URI location = uriInfo.getBaseUriBuilder().path(path).build();
-            return noCache(setCommonHeaders(Response.created(location), protocolVersion, transactionManager)
+            final URI transactionUri = buildResourceUri("data-transfer", portType, portId, "transactions", transactionId);
+
+            return noCache(setCommonHeaders(Response.created(transactionUri), protocolVersion, transactionManager)
                     .header(LOCATION_URI_INTENT_NAME, LOCATION_URI_INTENT_VALUE))
                     .entity(entity).build();
         }

http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/DataTransferResource.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/DataTransferResource.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/DataTransferResource.java
index bf55c05..1afedad 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/DataTransferResource.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/DataTransferResource.java
@@ -48,7 +48,6 @@ import org.apache.nifi.remote.protocol.HandshakeProperty;
 import org.apache.nifi.remote.protocol.ResponseCode;
 import org.apache.nifi.remote.protocol.http.HttpFlowFileServerProtocol;
 import org.apache.nifi.remote.protocol.http.StandardHttpFlowFileServerProtocol;
-import org.apache.nifi.stream.io.ByteArrayOutputStream;
 import org.apache.nifi.util.NiFiProperties;
 import org.apache.nifi.web.NiFiServiceFacade;
 import org.apache.nifi.web.api.entity.TransactionResultEntity;
@@ -74,6 +73,7 @@ import javax.ws.rs.core.MediaType;
 import javax.ws.rs.core.Response;
 import javax.ws.rs.core.StreamingOutput;
 import javax.ws.rs.core.UriInfo;
+import java.io.ByteArrayOutputStream;
 import java.io.IOException;
 import java.io.InputStream;
 import java.io.OutputStream;

http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/SiteToSiteResource.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/SiteToSiteResource.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/SiteToSiteResource.java
index 36cfda1..6737936 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/SiteToSiteResource.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/SiteToSiteResource.java
@@ -30,9 +30,12 @@ import org.apache.nifi.cluster.coordination.ClusterCoordinator;
 import org.apache.nifi.cluster.coordination.node.NodeWorkload;
 import org.apache.nifi.cluster.protocol.NodeIdentifier;
 import org.apache.nifi.remote.HttpRemoteSiteListener;
+import org.apache.nifi.remote.PeerDescription;
+import org.apache.nifi.remote.PeerDescriptionModifier;
 import org.apache.nifi.remote.VersionNegotiator;
 import org.apache.nifi.remote.client.http.TransportProtocolVersionNegotiator;
 import org.apache.nifi.remote.exception.BadRequestException;
+import org.apache.nifi.remote.protocol.SiteToSiteTransportProtocol;
 import org.apache.nifi.remote.protocol.http.HttpHeaders;
 import org.apache.nifi.util.NiFiProperties;
 import org.apache.nifi.web.NiFiServiceFacade;
@@ -56,6 +59,8 @@ import java.io.IOException;
 import java.net.InetAddress;
 import java.net.UnknownHostException;
 import java.util.ArrayList;
+import java.util.Enumeration;
+import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
 
@@ -80,9 +85,11 @@ public class SiteToSiteResource extends ApplicationResource {
     private final ResponseCreator responseCreator = new ResponseCreator();
     private final VersionNegotiator transportProtocolVersionNegotiator = new TransportProtocolVersionNegotiator(1);
     private final HttpRemoteSiteListener transactionManager;
+    private final PeerDescriptionModifier peerDescriptionModifier;
 
     public SiteToSiteResource(final NiFiProperties nifiProperties) {
         transactionManager = HttpRemoteSiteListener.getInstance(nifiProperties);
+        peerDescriptionModifier = new PeerDescriptionModifier(nifiProperties);
     }
 
     /**
@@ -131,6 +138,34 @@ public class SiteToSiteResource extends ApplicationResource {
         // get the controller dto
         final ControllerDTO controller = serviceFacade.getSiteToSiteDetails();
 
+        // Alter s2s port.
+        final boolean modificationNeededRaw = peerDescriptionModifier.isModificationNeeded(SiteToSiteTransportProtocol.RAW);
+        final boolean modificationNeededHttp = peerDescriptionModifier.isModificationNeeded(SiteToSiteTransportProtocol.HTTP);
+        if (modificationNeededRaw || modificationNeededHttp) {
+            final PeerDescription source = getSourcePeerDescription(req);
+            final Boolean isSiteToSiteSecure = controller.isSiteToSiteSecure();
+            final String siteToSiteHostname = getSiteToSiteHostname(req);
+            final Map<String, String> httpHeaders = getHttpHeaders(req);
+
+            if (modificationNeededRaw) {
+                final PeerDescription rawTarget = new PeerDescription(siteToSiteHostname, controller.getRemoteSiteListeningPort(), isSiteToSiteSecure);
+                final PeerDescription modifiedRawTarget = peerDescriptionModifier.modify(source, rawTarget,
+                        SiteToSiteTransportProtocol.RAW, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>(httpHeaders));
+                controller.setRemoteSiteListeningPort(modifiedRawTarget.getPort());
+            }
+
+            if (modificationNeededHttp) {
+                final PeerDescription httpTarget = new PeerDescription(siteToSiteHostname, controller.getRemoteSiteHttpListeningPort(), isSiteToSiteSecure);
+                final PeerDescription modifiedHttpTarget = peerDescriptionModifier.modify(source, httpTarget,
+                        SiteToSiteTransportProtocol.HTTP, PeerDescriptionModifier.RequestType.SiteToSiteDetail, new HashMap<>(httpHeaders));
+                controller.setRemoteSiteHttpListeningPort(modifiedHttpTarget.getPort());
+                if (!controller.isSiteToSiteSecure() && modifiedHttpTarget.isSecure()) {
+                    // In order to enable TLS terminate at the reverse proxy server, even if NiFi itself is not secured, introduce the endpoint as secure.
+                    controller.setSiteToSiteSecure(true);
+                }
+            }
+        }
+
         // build the response entity
         final ControllerEntity entity = new ControllerEntity();
         entity.setController(controller);
@@ -147,6 +182,20 @@ public class SiteToSiteResource extends ApplicationResource {
         return noCache(Response.ok(entity)).build();
     }
 
+    private PeerDescription getSourcePeerDescription(@Context HttpServletRequest req) {
+        return new PeerDescription(req.getRemoteHost(), req.getRemotePort(), req.isSecure());
+    }
+
+    private Map<String, String> getHttpHeaders(@Context HttpServletRequest req) {
+        final Map<String, String> headers = new HashMap<>();
+        final Enumeration<String> headerNames = req.getHeaderNames();
+        while (headerNames.hasMoreElements()) {
+            final String name = headerNames.nextElement();
+            headers.put(name, req.getHeader(name));
+        }
+        return headers;
+    }
+
     /**
      * Returns the available Peers and its status of this NiFi.
      *
@@ -187,18 +236,29 @@ public class SiteToSiteResource extends ApplicationResource {
         }
 
         final List<PeerDTO> peers = new ArrayList<>();
+        final PeerDescription source = getSourcePeerDescription(req);
+        final boolean modificationNeeded = peerDescriptionModifier.isModificationNeeded(SiteToSiteTransportProtocol.HTTP);
+        final Map<String, String> headers = modificationNeeded ? getHttpHeaders(req) : null;
         if (properties.isNode()) {
 
             try {
                 final Map<NodeIdentifier, NodeWorkload> clusterWorkload = clusterCoordinator.getClusterWorkload();
-                clusterWorkload.entrySet().stream().forEach(entry -> {
+                clusterWorkload.forEach((nodeId, workload) -> {
+                    final String siteToSiteHostname = nodeId.getSiteToSiteAddress() == null ? nodeId.getApiAddress() : nodeId.getSiteToSiteAddress();
+                    final int siteToSitePort = nodeId.getSiteToSiteHttpApiPort() == null ? nodeId.getApiPort() : nodeId.getSiteToSiteHttpApiPort();
+
+                    PeerDescription target = new PeerDescription(siteToSiteHostname, siteToSitePort, nodeId.isSiteToSiteSecure());
+
+                    if (modificationNeeded) {
+                        target = peerDescriptionModifier.modify(source, target,
+                                SiteToSiteTransportProtocol.HTTP, PeerDescriptionModifier.RequestType.Peers, new HashMap<>(headers));
+                    }
+
                     final PeerDTO peer = new PeerDTO();
-                    final NodeIdentifier nodeId = entry.getKey();
-                    final String siteToSiteAddress = nodeId.getSiteToSiteAddress();
-                    peer.setHostname(siteToSiteAddress == null ? nodeId.getApiAddress() : siteToSiteAddress);
-                    peer.setPort(nodeId.getSiteToSiteHttpApiPort() == null ? nodeId.getApiPort() : nodeId.getSiteToSiteHttpApiPort());
-                    peer.setSecure(nodeId.isSiteToSiteSecure());
-                    peer.setFlowFileCount(entry.getValue().getFlowFileCount());
+                    peer.setHostname(target.getHostname());
+                    peer.setPort(target.getPort());
+                    peer.setSecure(target.isSecure());
+                    peer.setFlowFileCount(workload.getFlowFileCount());
                     peers.add(peer);
                 });
             } catch (IOException e) {
@@ -208,24 +268,20 @@ public class SiteToSiteResource extends ApplicationResource {
         } else {
             // Standalone mode.
             final PeerDTO peer = new PeerDTO();
+            final String siteToSiteHostname = getSiteToSiteHostname(req);
 
-            // Private IP address or hostname may not be accessible from client in some environments.
-            // So, use the value defined in nifi.properties instead when it is defined.
-            final String remoteInputHost = properties.getRemoteInputHost();
-            String localName;
-            try {
-                // Get local host name using InetAddress if available, same as RAW socket does.
-                localName = InetAddress.getLocalHost().getHostName();
-            } catch (UnknownHostException e) {
-                if (logger.isDebugEnabled()) {
-                    logger.debug("Failed to get local host name using InetAddress.", e);
-                }
-                localName = req.getLocalName();
+
+            PeerDescription target = new PeerDescription(siteToSiteHostname,
+                    properties.getRemoteInputHttpPort(), properties.isSiteToSiteSecure());
+
+            if (modificationNeeded) {
+                target = peerDescriptionModifier.modify(source, target,
+                        SiteToSiteTransportProtocol.HTTP, PeerDescriptionModifier.RequestType.Peers, new HashMap<>(headers));
             }
 
-            peer.setHostname(isEmpty(remoteInputHost) ? localName : remoteInputHost);
-            peer.setPort(properties.getRemoteInputHttpPort());
-            peer.setSecure(properties.isSiteToSiteSecure());
+            peer.setHostname(target.getHostname());
+            peer.setPort(target.getPort());
+            peer.setSecure(target.isSecure());
             peer.setFlowFileCount(0);  // doesn't matter how many FlowFiles we have, because we're the only host.
 
             peers.add(peer);
@@ -237,6 +293,24 @@ public class SiteToSiteResource extends ApplicationResource {
         return noCache(setCommonHeaders(Response.ok(entity), transportProtocolVersion, transactionManager)).build();
     }
 
+    private String getSiteToSiteHostname(final HttpServletRequest req) {
+        // Private IP address or hostname may not be accessible from client in some environments.
+        // So, use the value defined in nifi.properties instead when it is defined.
+        final String remoteInputHost = properties.getRemoteInputHost();
+        String localName;
+        try {
+            // Get local host name using InetAddress if available, same as RAW socket does.
+            localName = InetAddress.getLocalHost().getHostName();
+        } catch (UnknownHostException e) {
+            if (logger.isDebugEnabled()) {
+                logger.debug("Failed to get local host name using InetAddress.", e);
+            }
+            localName = req.getLocalName();
+        }
+
+        return isEmpty(remoteInputHost) ? localName : remoteInputHost;
+    }
+
     // setters
 
     public void setServiceFacade(final NiFiServiceFacade serviceFacade) {

http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/test/java/org/apache/nifi/web/api/TestDataTransferResource.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/test/java/org/apache/nifi/web/api/TestDataTransferResource.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/test/java/org/apache/nifi/web/api/TestDataTransferResource.java
index 3dac9ce..d16a8a6 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/test/java/org/apache/nifi/web/api/TestDataTransferResource.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/test/java/org/apache/nifi/web/api/TestDataTransferResource.java
@@ -41,10 +41,14 @@ import javax.ws.rs.core.StreamingOutput;
 import javax.ws.rs.core.UriBuilder;
 import javax.ws.rs.core.UriInfo;
 import java.io.InputStream;
+import java.lang.reflect.Field;
 import java.net.URI;
 import java.net.URISyntaxException;
 import java.net.URL;
 
+import static org.apache.nifi.web.api.ApplicationResource.PROXY_HOST_HTTP_HEADER;
+import static org.apache.nifi.web.api.ApplicationResource.PROXY_PORT_HTTP_HEADER;
+import static org.apache.nifi.web.api.ApplicationResource.PROXY_SCHEME_HTTP_HEADER;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertTrue;
 import static org.mockito.Matchers.any;
@@ -53,6 +57,7 @@ import static org.mockito.Mockito.doAnswer;
 import static org.mockito.Mockito.doReturn;
 import static org.mockito.Mockito.doThrow;
 import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.when;
 
 public class TestDataTransferResource {
 
@@ -156,6 +161,17 @@ public class TestDataTransferResource {
 
         final ServletContext context = null;
         final UriInfo uriInfo = mockUriInfo(locationUriStr);
+        final Field uriInfoField = resource.getClass().getSuperclass().getSuperclass()
+                .getDeclaredField("uriInfo");
+        uriInfoField.setAccessible(true);
+        uriInfoField.set(resource, uriInfo);
+
+        final HttpServletRequest request = mock(HttpServletRequest.class);
+        final Field httpServletRequestField = resource.getClass().getSuperclass().getSuperclass()
+                .getDeclaredField("httpServletRequest");
+        httpServletRequestField.setAccessible(true);
+        httpServletRequestField.set(resource, request);
+
         final InputStream inputStream = null;
 
        final Response response = resource.createPortTransaction("input-ports", "port-id", req, context, uriInfo, inputStream);
@@ -168,6 +184,41 @@ public class TestDataTransferResource {
     }
 
     @Test
+    public void testCreateTransactionThroughReverseProxy() throws Exception {
+        final HttpServletRequest req = createCommonHttpServletRequest();
+
+        final DataTransferResource resource = getDataTransferResource();
+
+        final String locationUriStr = "https://nifi2.example.com:443/nifi-api/data-transfer/input-ports/port-id/transactions/transaction-id";
+
+        final ServletContext context = null;
+        final UriInfo uriInfo = mockUriInfo(locationUriStr);
+        final Field uriInfoField = resource.getClass().getSuperclass().getSuperclass()
+                .getDeclaredField("uriInfo");
+        uriInfoField.setAccessible(true);
+        uriInfoField.set(resource, uriInfo);
+
+        final HttpServletRequest request = mock(HttpServletRequest.class);
+        when(request.getHeader(PROXY_SCHEME_HTTP_HEADER)).thenReturn("https");
+        when(request.getHeader(PROXY_HOST_HTTP_HEADER)).thenReturn("nifi2.example.com");
+        when(request.getHeader(PROXY_PORT_HTTP_HEADER)).thenReturn("443");
+        final Field httpServletRequestField = resource.getClass().getSuperclass().getSuperclass()
+                .getDeclaredField("httpServletRequest");
+        httpServletRequestField.setAccessible(true);
+        httpServletRequestField.set(resource, request);
+
+        final InputStream inputStream = null;
+
+        final Response response = resource.createPortTransaction("input-ports", "port-id", req, context, uriInfo, inputStream);
+
+        TransactionResultEntity resultEntity = (TransactionResultEntity) response.getEntity();
+
+        assertEquals(201, response.getStatus());
+        assertEquals(ResponseCode.PROPERTIES_OK.getCode(), resultEntity.getResponseCode());
+        assertEquals(locationUriStr, response.getMetadata().getFirst(HttpHeaders.LOCATION_HEADER_NAME).toString());
+    }
+
+    @Test
     public void testExtendTransaction() throws Exception {
         final HttpServletRequest req = createCommonHttpServletRequest();
 


[4/5] nifi git commit: NIFI-4932: Enable S2S work behind a Reverse Proxy Adding S2S endpoint Reverse Proxy mapping capability. Added license header to SVG files. Incorporated review comments. Use regex to check property key processing. Catch AttributeExp

Posted by mc...@apache.org.
http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-docs/src/main/asciidoc/images/s2s-rproxy-http.svg
----------------------------------------------------------------------
diff --git a/nifi-docs/src/main/asciidoc/images/s2s-rproxy-http.svg b/nifi-docs/src/main/asciidoc/images/s2s-rproxy-http.svg
new file mode 100644
index 0000000..c845aae
--- /dev/null
+++ b/nifi-docs/src/main/asciidoc/images/s2s-rproxy-http.svg
@@ -0,0 +1,17 @@
+<?xml version="1.0" standalone="yes"?>
+<!--
+  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 version="1.1" viewBox="0.0 0.0 800.0 450.0" fill="none" stroke="none" stroke-linecap="square" stroke-miterlimit="10" xmlns="http://www.w3.org/2000/svg" xmlns:xlink="http://www.w3.org/1999/xlink"><clipPath id="p.0"><path d="m0 0l800.0 0l0 600.0l-800.0 0l0 -600.0z" clip-rule="nonzero"></path></clipPath><g clip-path="url(#p.0)"><path fill="#000000" fill-opacity="0.0" d="m0 0l800.0 0l0 600.0l-800.0 0z" fill-rule="evenodd"></path><path fill="#000000" fill-opacity="0.0" d="m85.99213 13.036745l214.99213 0l0 34.3937l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m96.05463 37.396748l0 -11.453127l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.8593769l0 5.265625l-1.40625 0l0 -5.265625q0 -1.0468769 -0.453125 -1.5312519q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.5156269l0 4.546875l-1.4
 0625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765627l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.843752q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765627l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.843752q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484377l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.5
 46875q0.421875 0.984375 0.421875 2.171877q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.5937519 -0.65625 -2.375002q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.437502zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5000019 -0.296875 -1.1093769q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.2812
 5q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.3750019q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 -4.218752l0 -1.59375l1.59375 0l0 1.59375l-1.59375 0zm0 6.703127l0 -1.609375l1.59375 0l0 1.609375l-1.59375 0zm2.9921875 0.203125l3.328125 -11.859377l1.125 0l-3.3125 11.859377l-1.140625 0zm4.4453125 0l3.328125 -11.859377l1.125 0l-3.3125 11.859377l-1.140625 0zm5.5078125 -0.203125l0 -8.296877l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0
 .4375 1.046875q0.078125 0.390625 0.078125 1.3593769l0 5.109375l-1.40625 0l0 -5.046875q0 -0.8593769 -0.171875 -1.2812519q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.156252l0 4.53125l-1.40625 0zm8.8984375 -9.843752l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843752l0 -8.296877l1.40625 0l0 8.296877l-1.40625 0zm3.8828125 0l0 -7.203127l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203127l-1.40625 0zm4.1171875 -9.843752l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843752l0 -8.296877l1.40625 0l0 8.296877l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.7265625 -2.671875l1.453125 0.17187
 5q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.093752 1.078125 -3.250002q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171877q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.0468769 -0.53125 -1.5625019q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.7187519zm6.8984375 4.953125l3.03125 -4.3125l-2.8125 -3.984377l1.765625 0l1.265625 1.9375q0.359375 0.5625 0.578125 0.9375q0.34375 -0.515625 0.640625 -0.921875l1.390625 -1.953125l1.6875 0l-2.875 3.906252l3.09375 4.390625l-1.734375 0l-1.703125 -2.578125l-0.453125 -0.703125l-2.171875 3.28125l-1.703125 0zm14.34375 -1.03125q-0.78125 0.671875 -1.5 0.953125q-0.71875 0.265625 -1.546875 0.265625q-1.375 0 -2.109375 -0.67187
 5q-0.734375 -0.671875 -0.734375 -1.703125q0 -0.609375 0.28125 -1.109375q0.28125 -0.515625 0.71875 -0.8125q0.453125 -0.3125 1.015625 -0.46875q0.421875 -0.109375 1.25 -0.203125q1.703125 -0.203125 2.515625 -0.484375q0 -0.2968769 0 -0.3750019q0 -0.859375 -0.390625 -1.203125q-0.546875 -0.484375 -1.609375 -0.484375q-0.984375 0 -1.46875 0.359375q-0.46875 0.34375 -0.6875 1.21875l-1.375 -0.1875q0.1875 -0.875 0.609375 -1.421875q0.4375 -0.546875 1.25 -0.828125q0.8125 -0.296875 1.875 -0.296875q1.0625 0 1.71875 0.25q0.671875 0.25 0.984375 0.625q0.3125 0.375 0.4375 0.953125q0.078125 0.359375 0.078125 1.2968769l0 1.875q0 1.96875 0.078125 2.484375q0.09375 0.515625 0.359375 1.0l-1.46875 0q-0.21875 -0.4375 -0.28125 -1.03125zm-0.109375 -3.140625q-0.765625 0.3125 -2.296875 0.53125q-0.875 0.125 -1.234375 0.28125q-0.359375 0.15625 -0.5625 0.46875q-0.1875 0.296875 -0.1875 0.65625q0 0.5625 0.421875 0.9375q0.4375 0.375 1.25 0.375q0.8125 0 1.4375 -0.34375q0.640625 -0.359375 0.9375 -0.984375q0.234375 -0.46875
  0.234375 -1.40625l0 -0.515625zm3.6015625 4.171875l0 -8.296877l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703127l-1.390625 0l0 -5.234375q0 -0.8437519 -0.140625 -1.2031269q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.8593769l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375019 -0.34375 -1.4062519q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.7187519l0 4.3125l-1.40625 0zm13.328125 3.1875l0 -11.484377l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171877q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.32812
 5 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.5937519 -0.65625 -2.375002q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.437502zm7.6015625 4.109375l0 -11.453127l1.40625 0l0 11.453127l-1.40625 0zm9.2578125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.093752 1.078125 -3.250002q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171877q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.0468769 -0.53125 -1.5
 625019q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.7187519zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.4609375 -3.046875l1.390625 0.1875q-0.234375 1.421875 -1.171875 2.234375q-0.921875 0.8125 -2.28125 0.8125q-1.703125 0 -2.75 -1.109375q-1.03125 -1.125 -1.03125 -3.203125q0 -1.3437519 0.4375 -2.343752q0.453125 -1.015625 1.359375 -1.515625q0.921875 -0.5 1.984375 -0.5q1.359375 0 2.21875 0.6875q0.859375 0.671875 1.09375 1.9375l-1.359375 0.203125q-0.203125 -0.828125 -0.703125 -1.25q-0.484375 -0.421875 -1.1875 -0.421875q-1.0625 0 -1.734375 0.765625q-0.65625 0.75 -0.65625 2.406252q0 1.671875 0.640625 2.4375q0.640625 0.75 1.671875 0.75q0.828125 0 1.375 -0.5q0.5625 -0.515625 0.703125 -1.578125zm2.0625 -1.109375q0 -2.296877 1.28125 -3.406252q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.093752q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.
 5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.5312519 -0.703125 -2.328127q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375002zm7.9765625 4.15625l0 -8.296877l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703127l-1.390625 0l0 -5.234375q0 -0.8437519 -0.140625 -1.2031269q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.8593769l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375019 -0.34375 -1.4062519q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.2187
 5 1.7187519l0 4.3125l-1.40625 0z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m476.09448 69.94226l214.99213 0l0 58.04724l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m486.15698 94.30226l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.82
 8125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm5.5859375 0l-2.546875 -8.296875l1.453125 0l1.328125 4.78125l0.484375 1.78125q0.03125 -0.125 0.4375 -1.703125l1.3125 -4.859375l1.453125 0l1.234375 4.8125l0.421875 1.578125l0.46875 -1.59375l1.421875 -4.796875l1.375 0l-2.59375 8.296875l-1.46875 0l-1.3125 -4.96875l-0.328125 -1.421875l-1.671875 6.390625l-1.46875 0zm15.6953125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.7031
 25 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm9.1328125 4.953125l-1.3125 0l0 -11.453125l1.40625 0l0 4.078125q0.890625 -1.109375 2.28125 -1.109375q0.765625 0 1.4375 0.3125q0.6875 0.296875 1.125 0.859375q0.453125 0.5625 0.703125 1.359375q0.25 0.78125 0.25 1.671875q0 2.140625 -1.0625 3.3125q-1.046875 1.15625 -2.53125 1.15625q-1.46875 0 -2.296875 -1.234375l0 1.046875zm-0.015625 -4.21875q0 1.5 0.40625 2.15625q0.65625 1.09375 1.796875 1.09375q0.921875 0 1.59375 -0.796875q0.671875 -0.8125 0.671875 -2.390625q0 -1.625 -0.65625 -2.390625q-0.640625 -0.78125 -1.546875 -0.78125q-0.921875 0 -1.59375 0.796875q-0.671875 0.796875 -0.671875 2.3125zm8.0078125 4.21875l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.1406
 25q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-
 1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 
 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 2.484375l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.14062
 5 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm8.3671875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.4140625 1.671875l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0
 .4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm11.625 1.21875l0.203125 1.25q-0.59375 0.12
 5 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm1.953125 3.265625l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.29687
 5l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.1640625 -5.65625q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640
 625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375z" fill-rule="nonzero"></path><path fill="#000000" d="m486.15698 113.30226l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.2
 03125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0390625 0l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm11.015625 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.18
 75 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm7.8359375 4.953125l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm12.796875 -4
 .15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm11.0390625 2.890625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm7.0546875 -1.40625l1.453125 0.171875q-0.
 34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.5546875 0l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 
 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm13.0703125 4.109375l0 -1.21875q-0.96875 1.40625 -2.640625 1.40625q-0.734375 0 -1.375 -0.28125q-0.625 -0.28125 -0.9375 -0.703125q-0.3125 -0.
 4375 -0.4375 -1.046875q-0.078125 -0.421875 -0.078125 -1.3125l0 -5.140625l1.40625 0l0 4.59375q0 1.109375 0.078125 1.484375q0.140625 0.5625 0.5625 0.875q0.4375 0.3125 1.0625 0.3125q0.640625 0 1.1875 -0.3125q0.5625 -0.328125 0.78125 -0.890625q0.234375 -0.5625 0.234375 -1.625l0 -4.4375l1.40625 0l0 8.296875l-1.25 0zm6.5234375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.7734375 1.265625l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625
 q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm8.3671875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.4140625 1.671875l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625
  -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm11.625 1.21875l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -
 1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm1.953125 3.265625l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.8
 90625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.1640625 -5.65625q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546
 875 0.828125q-0.734375 1.046875 -0.734375 3.859375z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m392.0 4.553806l214.99213 0l0 58.047245l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m402.0625 28.913807l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375
  -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm5.5859375 0l-2.546875 -8.296875l1.453125 0l1.328125 4.78125l0.484375 1.78125q0.03125 -0.125 0.4375 -1.703125l1.3125 -4.859375l1.453125 0l1.234375 4.8125l0.421875 1.578125l0.46875 -1.59375l1.421875 -4.796875l1.375 0l-2.59375 8.296875l-1.46875 0l-1.3125 -4.96875l-0.328125 -1.421875l-1.671875 6.390625l-1.46875 0zm15.6953125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625
  2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm9.1328125 4.953125l-1.3125 0l0 -11.453125l1.40625 0l0 4.078125q0.890625 -1.109375 2.28125 -1.109375q0.765625 0 1.4375 0.3125q0.6875 0.296875 1.125 0.859375q0.453125 0.5625 0.703125 1.359375q0.25 0.78125 0.25 1.671875q0 2.140625 -1.0625 3.3125q-1.046875 1.15625 -2.53125 1.15625q-1.46875 0 -2.296875 -1.234375l0 1.046875zm-0.015625 -4.21875q0 1.5 0.40625 2.15625q0.65625 1.09375 1.796875 1.09375q0.921875 0 1.59375 -0.796875q0.671875 -0.8125 0.671875 -2.390625q0 -1.625 -0.65625 -2.390625q-0.640625 -0.78125 -1.546875 -0.78125q-0.921875 0 -1.59375 0.796875q-0.671875 0.796875 -0.671875 2.3125zm8.0078125 4.21875l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 
 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.406
 25 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0
 .46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 2.484375l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 3.1875l0 -11.484375l1.28125 0l0 1.078
 125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.1015625 -0.046875q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.7968
 75q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.9609375 4.15625l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm8.40625 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-
 7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm3.71875 -2.953125q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q
 0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.5
 15625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.046875 -2.765625 1.046875q-1.5 0 -2.5 -0.890625q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#000000" d="m402.0625 47.913807l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843
 75l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0390625 0l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm11.015625 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125
  1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm7.8359375 4.953125l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125
  0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm12.796875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm11.0390625 2.890625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.0937
 5l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm7.0546875 -1.40625l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l
 -1.40625 0zm3.5546875 0l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.
 65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm13.0703125 4.109375l0 -1.21875q-0.96875 1.40625 -2.640625 1.40625q-0.734375 0 -1.375 -0.28125q-0.625 -0.28125 -0.9375 -0.703125q-0.3125 -0.4375 -0.4375 -1.046875q-0.078125 -0.421875 -0.078125 -1.3125l0 -5.140625l1.40625 0l0 4.59375q0 1.109375 0.078125 1.484375q0.140625 0.5625 0.5625 0.875q0.4375 0.3125 1.0625 0.3125q0.640625 0 1.1875 -0.3125q0.5625 -0.328125 0.78125 -0.890625q0.234375 -0.5625 0.234375 -1.625l0 -4.4375l1.40625 0l0 8.296875l-1.25 0zm6.5234375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.7734375 1.265625l0 -1.6
 09375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.1015625 -0.046875q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125
  -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.9609375 4.15625l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm8.40625 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0
  0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm3.71875 -2.953125q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.2
 96875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.8
 90625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm12.773
 4375 3.3125l-1.40625 0l0 -8.96875q-0.515625 0.484375 -1.34375 0.96875q-0.8125 0.484375 -1.46875 0.734375l0 -1.359375q1.171875 -0.5625 2.046875 -1.34375q0.890625 -0.796875 1.265625 -1.53125l0.90625 0l0 11.5z" fill-rule="nonzero"></path><path fill="#ffffff" d="m59.0 61.866142l232.0 0l0 229.13385l-232.0 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m59.0 61.866142l232.0 0l0 229.13385l-232.0 0z" fill-rule="evenodd"></path><path fill="#000000" d="m69.46875 88.78614l0 -13.359375l5.921875 0q1.78125 0 2.703125 0.359375q0.9375 0.359375 1.484375 1.28125q0.5625 0.90625 0.5625 2.015625q0 1.40625 -0.921875 2.390625q-0.921875 0.96875 -2.84375 1.234375q0.703125 0.34375 1.078125 0.671875q0.765625 0.703125 1.453125 1.765625l2.328125 3.640625l-2.21875 0l-1.765625 -2.78125q-0.78125 -1.203125 -1.28125 -1.828125q-0.5 -0.640625 -0.90625 -0.890625q-0.390625 -0.265625 -0.796875 -0.359375q-0.296875 -0.078125 -0.984375 -0.078125l-2.0
 46875 0l0 5.9375l-1.765625 0zm1.765625 -7.453125l3.796875 0q1.21875 0 1.890625 -0.25q0.6875 -0.265625 1.046875 -0.8125q0.359375 -0.546875 0.359375 -1.1875q0 -0.953125 -0.6875 -1.5625q-0.6875 -0.609375 -2.1875 -0.609375l-4.21875 0l0 4.421875zm18.097946 4.34375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm11.828842 5.765625l-3.6875 -9.671875l1.734375 0l2.078125 5.796875q0.328125 0.9375 0.625 1.9375q0.203125 -0.765625 0.609375 -1.828125l2.1406
 25 -5.90625l1.6875 0l-3.65625 9.671875l-1.53125 0zm13.265625 -3.109375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm9.125717 5.765625l0 -9.671875l1.46875 0l0 1.46875q0.5625 -1.03125 1.03125 -1.359375q0.484375 -0.328125 1.0625 -0.328125q0.828125 0 1.6875 0.53125l-0.5625 1.515625q-0.609375 -0.359375 -1.203125 -0.359375q-0.546875 0 -0.96875 0.328125q-0.421875 0.328125 -0.609375 0.890625q-0.28125 0.875 -0.28125 1.921875l0 5.0625l-1.625 0zm5.572
 052 -2.890625l1.625 -0.25q0.125 0.96875 0.75 1.5q0.625 0.515625 1.75 0.515625q1.125 0 1.671875 -0.453125q0.546875 -0.46875 0.546875 -1.09375q0 -0.546875 -0.484375 -0.875q-0.328125 -0.21875 -1.671875 -0.546875q-1.8125 -0.46875 -2.515625 -0.796875q-0.6875 -0.328125 -1.046875 -0.90625q-0.359375 -0.59375 -0.359375 -1.3125q0 -0.640625 0.296875 -1.1875q0.296875 -0.5625 0.8125 -0.921875q0.375 -0.28125 1.03125 -0.46875q0.671875 -0.203125 1.421875 -0.203125q1.140625 0 2.0 0.328125q0.859375 0.328125 1.265625 0.890625q0.421875 0.5625 0.578125 1.5l-1.609375 0.21875q-0.109375 -0.75 -0.640625 -1.171875q-0.515625 -0.421875 -1.46875 -0.421875q-1.140625 0 -1.625 0.375q-0.46875 0.375 -0.46875 0.875q0 0.3125 0.1875 0.578125q0.203125 0.265625 0.640625 0.4375q0.234375 0.09375 1.4375 0.421875q1.75 0.453125 2.4375 0.75q0.6875 0.296875 1.078125 0.859375q0.390625 0.5625 0.390625 1.40625q0 0.828125 -0.484375 1.546875q-0.46875 0.71875 -1.375 1.125q-0.90625 0.390625 -2.046875 0.390625q-1.875 0 -2.875 -0.78125q
 -0.984375 -0.78125 -1.25 -2.328125zm16.609375 -0.21875l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm14.527771 5.765625l0 -13.359375l5.046875 0q1.328125 0 2.03125 0.125q0.96875 0.171875 1.640625 0.640625q0.671875 0.453125 1.078125 1.28125q0.40625 0.828125 0.40625 1.828125q0 1.703125 -1.09375 2.890625q-1.078125 1.171875 -3.921875 1.171875l-3.421875 0l0 5.421875l-1.765625 0zm1.765625 -7.0l3.453125 0q1.71875 0 2.4375 -0.640625q0.71875 -0.640625
  0.71875 -1.796875q0 -0.84375 -0.421875 -1.4375q-0.421875 -0.59375 -1.125 -0.78125q-0.4375 -0.125 -1.640625 -0.125l-3.421875 0l0 4.78125zm10.459198 7.0l0 -9.671875l1.46875 0l0 1.46875q0.5625 -1.03125 1.03125 -1.359375q0.484375 -0.328125 1.0625 -0.328125q0.828125 0 1.6875 0.53125l-0.5625 1.515625q-0.609375 -0.359375 -1.203125 -0.359375q-0.546875 0 -0.96875 0.328125q-0.421875 0.328125 -0.609375 0.890625q-0.28125 0.875 -0.28125 1.921875l0 5.0625l-1.625 0zm5.618927 -4.84375q0 -2.6875 1.484375 -3.96875q1.25 -1.078125 3.046875 -1.078125q2.0 0 3.265625 1.3125q1.265625 1.296875 1.265625 3.609375q0 1.859375 -0.5625 2.9375q-0.5625 1.0625 -1.640625 1.65625q-1.0625 0.59375 -2.328125 0.59375q-2.03125 0 -3.28125 -1.296875q-1.25 -1.3125 -1.25 -3.765625zm1.6875 0q0 1.859375 0.796875 2.796875q0.8125 0.921875 2.046875 0.921875q1.21875 0 2.03125 -0.921875q0.8125 -0.9375 0.8125 -2.84375q0 -1.796875 -0.8125 -2.71875q-0.8125 -0.921875 -2.03125 -0.921875q-1.234375 0 -2.046875 0.921875q-0.796875 0.90625 -0
 .796875 2.765625zm8.203842 4.84375l3.53125 -5.03125l-3.265625 -4.640625l2.046875 0l1.484375 2.265625q0.421875 0.640625 0.671875 1.078125q0.40625 -0.59375 0.734375 -1.0625l1.640625 -2.28125l1.953125 0l-3.34375 4.546875l3.59375 5.125l-2.015625 0l-1.984375 -3.0l-0.515625 -0.8125l-2.546875 3.8125l-1.984375 0zm10.34375 3.71875l-0.1875 -1.53125q0.546875 0.140625 0.9375 0.140625q0.546875 0 0.875 -0.1875q0.328125 -0.171875 0.546875 -0.5q0.15625 -0.25 0.5 -1.21875q0.046875 -0.140625 0.140625 -0.40625l-3.671875 -9.6875l1.765625 0l2.015625 5.59375q0.390625 1.078125 0.703125 2.25q0.28125 -1.125 0.671875 -2.203125l2.078125 -5.640625l1.640625 0l-3.6875 9.828125q-0.59375 1.609375 -0.921875 2.203125q-0.4375 0.8125 -1.0 1.1875q-0.5625 0.375 -1.34375 0.375q-0.484375 0 -1.0625 -0.203125z" fill-rule="nonzero"></path><path fill="#ffffff" d="m3.0 11.866141l82.99213 0l0 30.362206l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt
 " d="m3.0 11.866141l82.99213 0l0 30.362206l-82.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m22.96875 29.279743l1.765625 0.453125q-0.5625 2.171875 -2.0 3.328125q-1.4375 1.140625 -3.53125 1.140625q-2.15625 0 -3.515625 -0.875q-1.34375 -0.890625 -2.0625 -2.546875q-0.703125 -1.671875 -0.703125 -3.59375q0 -2.078125 0.796875 -3.625q0.796875 -1.5625 2.265625 -2.359375q1.484375 -0.8125 3.25 -0.8125q2.0 0 3.359375 1.015625q1.375 1.015625 1.90625 2.875l-1.734375 0.40625q-0.46875 -1.453125 -1.359375 -2.109375q-0.875 -0.671875 -2.203125 -0.671875q-1.546875 0 -2.578125 0.734375q-1.03125 0.734375 -1.453125 1.984375q-0.421875 1.234375 -0.421875 2.5625q0 1.703125 0.5 2.96875q0.5 1.265625 1.546875 1.90625q1.046875 0.625 2.265625 0.625q1.484375 0 2.515625 -0.859375q1.03125 -0.859375 1.390625 -2.546875zm3.6916962 4.6875l0 -13.359375l1.640625 0l0 13.359375l-1.640625 0zm4.191696 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.64062
 5 0zm10.769821 -3.109375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm9.141342 5.765625l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65
 625 -0.75 2.515625l0 5.28125l-1.640625 0zm13.953842 -1.46875l0.234375 1.453125q-0.6875 0.140625 -1.234375 0.140625q-0.890625 0 -1.390625 -0.28125q-0.484375 -0.28125 -0.6875 -0.734375q-0.203125 -0.46875 -0.203125 -1.9375l0 -5.578125l-1.203125 0l0 -1.265625l1.203125 0l0 -2.390625l1.625 -0.984375l0 3.375l1.65625 0l0 1.265625l-1.65625 0l0 5.671875q0 0.6875 0.078125 0.890625q0.09375 0.203125 0.28125 0.328125q0.203125 0.109375 0.578125 0.109375q0.265625 0 0.71875 -0.0625zm7.323929 1.46875l-1.640625 0l0 -10.453125q-0.59375 0.5625 -1.5625 1.140625q-0.953125 0.5625 -1.71875 0.84375l0 -1.59375q1.375 -0.640625 2.40625 -1.5625q1.03125 -0.921875 1.453125 -1.78125l1.0625 0l0 13.40625z" fill-rule="nonzero"></path><path fill="#ffffff" d="m392.0 74.04724l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m392.0 74.04724l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path fill="#0
 00000" d="m402.23438 100.96724l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65625 -0.75 2.515625l0 5.28125l-1.640625 0zm10.375732 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm4.535431 0l0 -8.40625l-1.453125 0l0 -1.265625l1.453125 0l0 -1.03125q0 -0.96875 0.171875 -1.453125q0.234375 -0.640625 0.828125 -1.03125q0.59375 -0.390625 1.671875 -0.390625q0.6875 0 1.53125 0.15625l-0.25 1.4375q-0.5 -0.09375 -0.953125 -0.09375q-0.75 0 -1.0625 0.328125q-0.3125 0.3125 -0.3125 1.1875l0 0.890625l1.890625 0l0 1.265625l-1.890625 0l0 8.40625l-1.625 0zm4.792694 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -
 9.671875l1.640625 0l0 9.671875l-1.640625 0zm3.691681 -6.59375q0 -2.359375 0.484375 -3.796875q0.484375 -1.453125 1.4375 -2.234375q0.96875 -0.78125 2.421875 -0.78125q1.078125 0 1.890625 0.4375q0.8125 0.421875 1.328125 1.25q0.53125 0.8125 0.828125 1.984375q0.3125 1.15625 0.3125 3.140625q0 2.359375 -0.484375 3.8125q-0.484375 1.4375 -1.453125 2.234375q-0.953125 0.78125 -2.421875 0.78125q-1.921875 0 -3.03125 -1.390625q-1.3125 -1.671875 -1.3125 -5.4375zm1.671875 0q0 3.296875 0.765625 4.390625q0.78125 1.078125 1.90625 1.078125q1.140625 0 1.90625 -1.09375q0.765625 -1.09375 0.765625 -4.375q0 -3.296875 -0.765625 -4.375q-0.765625 -1.078125 -1.921875 -1.078125q-1.125 0 -1.796875 0.953125q-0.859375 1.21875 -0.859375 4.5z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m76.0 104.92913l202.99213 0l0 175.74802l-202.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m76.0 104.92913l202.99213 0l0 175.74802l-202.99213 0z" 
 fill-rule="evenodd"></path><path fill="#000000" d="m90.171875 129.28914l0 -2.7500076l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.7500076l-1.40625 0zm0 -4.0312576l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.0312576l0 -2.7500076l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.7500076l-1.40625 0zm0 -4.0312576l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.5156326 1.421875 0.5156326q0.984375 0 1.671875 -0.6875076q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 
 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.5468826q-1.09375 1.046875 -2.765625 1.046875q-1.5 0 -2.5 -0.890625q-1.0 -0.9062576 -1.140625 -2.3437576z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m245.36765 150.5958l82.31662 0l0 -28.472435l82.31329 0" fill-rule="evenodd"></path><path stroke="#ff0000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" stroke-dasharray="4.0,3.0" d="m245.36765 150.5958l82.31662 0l0 -28.472435l76.313324 0" fill-rule="evenodd"></path><path fill="#ff0000" stroke="#ff0000" stroke-width="1.0" stroke-linecap="butt" d="m403.9976 123.77509l4.538086 -1.6517334l-4.538086 -1.6517334z" fill-rule="evenodd"></path><path fill="#cfe2f3" d="m410.0 109.51181l57.007874 0l0 25.228348
 l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m410.0 109.51181l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m421.82812 121.767235q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 0.9999924 -2.734375 0.9999924q-1.6875 0 -2.734375 -0.9999924q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34
 375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625
 l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.0468674 -2.765625 1.0468674q-1.5 0 -2.5 -0.8906174q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#ffffff" d="m410.0 141.58267l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m410.0 141.58267l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill=
 "#000000" d="m421.82812 153.8381q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.
 671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.
 796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm12.7734375 3.3125l-1.40625 0l0 -8.96875q-0.515625 0.484375 -1.34375 0.96875q-0.8125 0.484375 -1.46875 0.734375l0 -1.3
 59375q1.171875 -0.5625 2.046875 -1.34375q0.890625 -0.796875 1.265625 -1.53125l0.90625 0l0 11.5z" fill-rule="nonzero"></path><path fill="#ffffff" d="m470.0 135.81102l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m470.0 135.81102l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m480.23438 162.73102l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65625 -0.75 2.515625l0 5.28125l-1.640625 0zm10.375732 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm4.535431 0l0 -8.40625l-1.453
 125 0l0 -1.265625l1.453125 0l0 -1.03125q0 -0.96875 0.171875 -1.453125q0.234375 -0.640625 0.828125 -1.03125q0.59375 -0.390625 1.671875 -0.390625q0.6875 0 1.53125 0.15625l-0.25 1.4375q-0.5 -0.09375 -0.953125 -0.09375q-0.75 0 -1.0625 0.328125q-0.3125 0.3125 -0.3125 1.1875l0 0.890625l1.890625 0l0 1.265625l-1.890625 0l0 8.40625l-1.625 0zm4.792694 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm9.863556 0l-1.640625 0l0 -10.453125q-0.59375 0.5625 -1.5625 1.140625q-0.953125 0.5625 -1.71875 0.84375l0 -1.59375q1.375 -0.640625 2.40625 -1.5625q1.03125 -0.921875 1.453125 -1.78125l1.0625 0l0 13.40625z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m484.10278 172.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m484.10278 172.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#00000
 0" d="m495.9309 184.60188q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q
 -0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40
 625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.046875 -2.765625 1.046875q-1.5 0 -2.5 -0.890625q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#ffffff" d="m484.10278 203.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m484.10278 203.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m495.9309 215.60188q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390
 625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 
 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -
 1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm12.7734375 3.3125l-1.40625 0l0 -8.96875q-0.515625 0.484375 -1.34375 0.96875q-0.8125 0.484375 -1.46875 0.734375l0 -1.359375q1.171875 -0.5625 2.046875 -1.34375q0.890625 -0.796875 1.265625 -1.53125l0.90625 0l0 11.5z" fill-rule="nonzero"></path><path fill="#ffffff" d="m552.0 190.81102l82.99213 0l0 96.9449l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" 
 stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m552.0 190.81102l82.99213 0l0 96.9449l-82.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m562.2344 217.73102l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65625 -0.75 2.515625l0 5.28125l-1.640625 0zm10.375732 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm4.5354004 0l0 -8.40625l-1.453125 0l0 -1.265625l1.453125 0l0 -1.03125q0 -0.96875 0.171875 -1.453125q0.234375 -0.640625 0.828125 -1.03125q0.59375 -0.390625 1.671875 -0.390625q0.6875 0 1.53125 0.15625l-0.25 1.4375q-0.5 -0.09375 -0.953125 -0.09375q-0.75 0 -1.0625 0.328125q-0.3125 0.3125
  -0.3125 1.1875l0 0.890625l1.890625 0l0 1.265625l-1.890625 0l0 8.40625l-1.625 0zm4.7927246 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm12.301025 -1.578125l0 1.578125l-8.828125 0q-0.015625 -0.59375 0.1875 -1.140625q0.34375 -0.90625 1.078125 -1.78125q0.75 -0.875 2.15625 -2.015625q2.171875 -1.78125 2.9375 -2.828125q0.765625 -1.046875 0.765625 -1.96875q0 -0.984375 -0.703125 -1.640625q-0.6875 -0.671875 -1.8125 -0.671875q-1.1875 0 -1.90625 0.71875q-0.703125 0.703125 -0.703125 1.953125l-1.6875 -0.171875q0.171875 -1.890625 1.296875 -2.875q1.140625 -0.984375 3.03125 -0.984375q1.921875 0 3.046875 1.0625q1.125 1.0625 1.125 2.640625q0 0.796875 -0.328125 1.578125q-0.328125 0.78125 -1.09375 1.640625q-0.75 0.84375 -2.53125 2.34375q-1.46875 1.234375 -1.890625 1.6875q-0.421875 0.4375 -0.6875 0.875l6.546875 0z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m570.00574 227.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule=
 "evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m570.00574 227.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m581.83386 239.60188q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1
 .234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21
 875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.046875 -2.765625 1.046875q-1.5 0 -2.5 -0.890625q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#ffffff" d="m570.00574 258.34647l57.007874 0l0 25.228333l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m570.00574 258.34647l57.007874 0l0 25.228333l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m581.83386 270.60187q-0
 .875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 
 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.42187

<TRUNCATED>

[3/5] nifi git commit: NIFI-4932: Enable S2S work behind a Reverse Proxy Adding S2S endpoint Reverse Proxy mapping capability. Added license header to SVG files. Incorporated review comments. Use regex to check property key processing. Catch AttributeExp

Posted by mc...@apache.org.
http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-docs/src/main/asciidoc/images/s2s-rproxy-portnumber.svg
----------------------------------------------------------------------
diff --git a/nifi-docs/src/main/asciidoc/images/s2s-rproxy-portnumber.svg b/nifi-docs/src/main/asciidoc/images/s2s-rproxy-portnumber.svg
new file mode 100644
index 0000000..47e3284
--- /dev/null
+++ b/nifi-docs/src/main/asciidoc/images/s2s-rproxy-portnumber.svg
@@ -0,0 +1,17 @@
+<?xml version="1.0" standalone="yes"?>
+<!--
+  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 version="1.1" viewBox="0.0 0.0 800.0 450.0" fill="none" stroke="none" stroke-linecap="square" stroke-miterlimit="10" xmlns="http://www.w3.org/2000/svg" xmlns:xlink="http://www.w3.org/1999/xlink"><clipPath id="p.0"><path d="m0 0l800.0 0l0 600.0l-800.0 0l0 -600.0z" clip-rule="nonzero"></path></clipPath><g clip-path="url(#p.0)"><path fill="#000000" fill-opacity="0.0" d="m0 0l800.0 0l0 600.0l-800.0 0z" fill-rule="evenodd"></path><path fill="#000000" fill-opacity="0.0" d="m85.99213 13.036745l214.99213 0l0 34.3937l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m96.05463 37.396748l0 -11.453127l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.8593769l0 5.265625l-1.40625 0l0 -5.265625q0 -1.0468769 -0.453125 -1.5312519q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.5156269l0 4.546875l-1.4
 0625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765627l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.843752q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765627l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.843752q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484377l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.5
 46875q0.421875 0.984375 0.421875 2.171877q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.5937519 -0.65625 -2.375002q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.437502zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5000019 -0.296875 -1.1093769q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.2812
 5q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.3750019q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 -4.218752l0 -1.59375l1.59375 0l0 1.59375l-1.59375 0zm0 6.703127l0 -1.609375l1.59375 0l0 1.609375l-1.59375 0zm2.9921875 0.203125l3.328125 -11.859377l1.125 0l-3.3125 11.859377l-1.140625 0zm4.4453125 0l3.328125 -11.859377l1.125 0l-3.3125 11.859377l-1.140625 0zm5.5078125 -0.203125l0 -8.296877l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0
 .4375 1.046875q0.078125 0.390625 0.078125 1.3593769l0 5.109375l-1.40625 0l0 -5.046875q0 -0.8593769 -0.171875 -1.2812519q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.156252l0 4.53125l-1.40625 0zm8.8984375 -9.843752l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843752l0 -8.296877l1.40625 0l0 8.296877l-1.40625 0zm3.8828125 0l0 -7.203127l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203127l-1.40625 0zm4.1171875 -9.843752l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843752l0 -8.296877l1.40625 0l0 8.296877l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.7265625 -2.671875l1.453125 0.17187
 5q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.093752 1.078125 -3.250002q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171877q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.0468769 -0.53125 -1.5625019q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.7187519zm6.8984375 4.953125l3.03125 -4.3125l-2.8125 -3.984377l1.765625 0l1.265625 1.9375q0.359375 0.5625 0.578125 0.9375q0.34375 -0.515625 0.640625 -0.921875l1.390625 -1.953125l1.6875 0l-2.875 3.906252l3.09375 4.390625l-1.734375 0l-1.703125 -2.578125l-0.453125 -0.703125l-2.171875 3.28125l-1.703125 0zm14.34375 -1.03125q-0.78125 0.671875 -1.5 0.953125q-0.71875 0.265625 -1.546875 0.265625q-1.375 0 -2.109375 -0.67187
 5q-0.734375 -0.671875 -0.734375 -1.703125q0 -0.609375 0.28125 -1.109375q0.28125 -0.515625 0.71875 -0.8125q0.453125 -0.3125 1.015625 -0.46875q0.421875 -0.109375 1.25 -0.203125q1.703125 -0.203125 2.515625 -0.484375q0 -0.2968769 0 -0.3750019q0 -0.859375 -0.390625 -1.203125q-0.546875 -0.484375 -1.609375 -0.484375q-0.984375 0 -1.46875 0.359375q-0.46875 0.34375 -0.6875 1.21875l-1.375 -0.1875q0.1875 -0.875 0.609375 -1.421875q0.4375 -0.546875 1.25 -0.828125q0.8125 -0.296875 1.875 -0.296875q1.0625 0 1.71875 0.25q0.671875 0.25 0.984375 0.625q0.3125 0.375 0.4375 0.953125q0.078125 0.359375 0.078125 1.2968769l0 1.875q0 1.96875 0.078125 2.484375q0.09375 0.515625 0.359375 1.0l-1.46875 0q-0.21875 -0.4375 -0.28125 -1.03125zm-0.109375 -3.140625q-0.765625 0.3125 -2.296875 0.53125q-0.875 0.125 -1.234375 0.28125q-0.359375 0.15625 -0.5625 0.46875q-0.1875 0.296875 -0.1875 0.65625q0 0.5625 0.421875 0.9375q0.4375 0.375 1.25 0.375q0.8125 0 1.4375 -0.34375q0.640625 -0.359375 0.9375 -0.984375q0.234375 -0.46875
  0.234375 -1.40625l0 -0.515625zm3.6015625 4.171875l0 -8.296877l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703127l-1.390625 0l0 -5.234375q0 -0.8437519 -0.140625 -1.2031269q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.8593769l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375019 -0.34375 -1.4062519q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.7187519l0 4.3125l-1.40625 0zm13.328125 3.1875l0 -11.484377l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171877q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.32812
 5 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.5937519 -0.65625 -2.375002q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.437502zm7.6015625 4.109375l0 -11.453127l1.40625 0l0 11.453127l-1.40625 0zm9.2578125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.093752 1.078125 -3.250002q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171877q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.0468769 -0.53125 -1.5
 625019q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.7187519zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.4609375 -3.046875l1.390625 0.1875q-0.234375 1.421875 -1.171875 2.234375q-0.921875 0.8125 -2.28125 0.8125q-1.703125 0 -2.75 -1.109375q-1.03125 -1.125 -1.03125 -3.203125q0 -1.3437519 0.4375 -2.343752q0.453125 -1.015625 1.359375 -1.515625q0.921875 -0.5 1.984375 -0.5q1.359375 0 2.21875 0.6875q0.859375 0.671875 1.09375 1.9375l-1.359375 0.203125q-0.203125 -0.828125 -0.703125 -1.25q-0.484375 -0.421875 -1.1875 -0.421875q-1.0625 0 -1.734375 0.765625q-0.65625 0.75 -0.65625 2.406252q0 1.671875 0.640625 2.4375q0.640625 0.75 1.671875 0.75q0.828125 0 1.375 -0.5q0.5625 -0.515625 0.703125 -1.578125zm2.0625 -1.109375q0 -2.296877 1.28125 -3.406252q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.093752q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.
 5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.5312519 -0.703125 -2.328127q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375002zm7.9765625 4.15625l0 -8.296877l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703127l-1.390625 0l0 -5.234375q0 -0.8437519 -0.140625 -1.2031269q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.8593769l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375019 -0.34375 -1.4062519q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.2187
 5 1.7187519l0 4.3125l-1.40625 0z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m476.09448 69.94226l214.99213 0l0 58.04724l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m486.15698 94.30226l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.82
 8125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm5.5859375 0l-2.546875 -8.296875l1.453125 0l1.328125 4.78125l0.484375 1.78125q0.03125 -0.125 0.4375 -1.703125l1.3125 -4.859375l1.453125 0l1.234375 4.8125l0.421875 1.578125l0.46875 -1.59375l1.421875 -4.796875l1.375 0l-2.59375 8.296875l-1.46875 0l-1.3125 -4.96875l-0.328125 -1.421875l-1.671875 6.390625l-1.46875 0zm15.6953125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.7031
 25 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm9.1328125 4.953125l-1.3125 0l0 -11.453125l1.40625 0l0 4.078125q0.890625 -1.109375 2.28125 -1.109375q0.765625 0 1.4375 0.3125q0.6875 0.296875 1.125 0.859375q0.453125 0.5625 0.703125 1.359375q0.25 0.78125 0.25 1.671875q0 2.140625 -1.0625 3.3125q-1.046875 1.15625 -2.53125 1.15625q-1.46875 0 -2.296875 -1.234375l0 1.046875zm-0.015625 -4.21875q0 1.5 0.40625 2.15625q0.65625 1.09375 1.796875 1.09375q0.921875 0 1.59375 -0.796875q0.671875 -0.8125 0.671875 -2.390625q0 -1.625 -0.65625 -2.390625q-0.640625 -0.78125 -1.546875 -0.78125q-0.921875 0 -1.59375 0.796875q-0.671875 0.796875 -0.671875 2.3125zm8.0078125 4.21875l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.1406
 25q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-
 1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 
 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 2.484375l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.14062
 5 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm8.3671875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.4140625 1.671875l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0
 .4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm11.625 1.21875l0.203125 1.25q-0.59375 0.12
 5 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm1.953125 3.265625l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.29687
 5l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.1640625 -5.65625q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640
 625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375z" fill-rule="nonzero"></path><path fill="#000000" d="m486.15698 113.30226l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.2
 03125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0390625 0l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm11.015625 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.18
 75 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm7.8359375 4.953125l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm12.796875 -4
 .15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm11.0390625 2.890625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm7.0546875 -1.40625l1.453125 0.171875q-0.
 34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.5546875 0l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 
 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm13.0703125 4.109375l0 -1.21875q-0.96875 1.40625 -2.640625 1.40625q-0.734375 0 -1.375 -0.28125q-0.625 -0.28125 -0.9375 -0.703125q-0.3125 -0.
 4375 -0.4375 -1.046875q-0.078125 -0.421875 -0.078125 -1.3125l0 -5.140625l1.40625 0l0 4.59375q0 1.109375 0.078125 1.484375q0.140625 0.5625 0.5625 0.875q0.4375 0.3125 1.0625 0.3125q0.640625 0 1.1875 -0.3125q0.5625 -0.328125 0.78125 -0.890625q0.234375 -0.5625 0.234375 -1.625l0 -4.4375l1.40625 0l0 8.296875l-1.25 0zm6.5234375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.7734375 1.265625l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625
 q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm8.3671875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.4140625 1.671875l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625
  -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm11.625 1.21875l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -
 1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm1.953125 3.265625l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.8
 90625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.1640625 -5.65625q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546
 875 0.828125q-0.734375 1.046875 -0.734375 3.859375z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m392.0 4.553806l214.99213 0l0 58.047245l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m402.0625 28.913807l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375
  -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm5.5859375 0l-2.546875 -8.296875l1.453125 0l1.328125 4.78125l0.484375 1.78125q0.03125 -0.125 0.4375 -1.703125l1.3125 -4.859375l1.453125 0l1.234375 4.8125l0.421875 1.578125l0.46875 -1.59375l1.421875 -4.796875l1.375 0l-2.59375 8.296875l-1.46875 0l-1.3125 -4.96875l-0.328125 -1.421875l-1.671875 6.390625l-1.46875 0zm15.6953125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625
  2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm9.1328125 4.953125l-1.3125 0l0 -11.453125l1.40625 0l0 4.078125q0.890625 -1.109375 2.28125 -1.109375q0.765625 0 1.4375 0.3125q0.6875 0.296875 1.125 0.859375q0.453125 0.5625 0.703125 1.359375q0.25 0.78125 0.25 1.671875q0 2.140625 -1.0625 3.3125q-1.046875 1.15625 -2.53125 1.15625q-1.46875 0 -2.296875 -1.234375l0 1.046875zm-0.015625 -4.21875q0 1.5 0.40625 2.15625q0.65625 1.09375 1.796875 1.09375q0.921875 0 1.59375 -0.796875q0.671875 -0.8125 0.671875 -2.390625q0 -1.625 -0.65625 -2.390625q-0.640625 -0.78125 -1.546875 -0.78125q-0.921875 0 -1.59375 0.796875q-0.671875 0.796875 -0.671875 2.3125zm8.0078125 4.21875l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 
 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.406
 25 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0
 .46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 2.484375l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 3.1875l0 -11.484375l1.28125 0l0 1.078
 125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.1015625 -0.046875q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.7968
 75q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.9609375 4.15625l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm8.40625 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-
 7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm3.71875 -2.953125q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q
 0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.5
 15625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.046875 -2.765625 1.046875q-1.5 0 -2.5 -0.890625q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#000000" d="m402.0625 47.913807l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843
 75l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0390625 0l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm11.015625 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125
  1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm7.8359375 4.953125l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125
  0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm12.796875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm11.0390625 2.890625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.0937
 5l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm7.0546875 -1.40625l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l
 -1.40625 0zm3.5546875 0l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.
 65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm13.0703125 4.109375l0 -1.21875q-0.96875 1.40625 -2.640625 1.40625q-0.734375 0 -1.375 -0.28125q-0.625 -0.28125 -0.9375 -0.703125q-0.3125 -0.4375 -0.4375 -1.046875q-0.078125 -0.421875 -0.078125 -1.3125l0 -5.140625l1.40625 0l0 4.59375q0 1.109375 0.078125 1.484375q0.140625 0.5625 0.5625 0.875q0.4375 0.3125 1.0625 0.3125q0.640625 0 1.1875 -0.3125q0.5625 -0.328125 0.78125 -0.890625q0.234375 -0.5625 0.234375 -1.625l0 -4.4375l1.40625 0l0 8.296875l-1.25 0zm6.5234375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.7734375 1.265625l0 -1.6
 09375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.1015625 -0.046875q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125
  -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.9609375 4.15625l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm8.40625 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0
  0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm3.71875 -2.953125q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.2
 96875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.8
 90625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm12.773
 4375 3.3125l-1.40625 0l0 -8.96875q-0.515625 0.484375 -1.34375 0.96875q-0.8125 0.484375 -1.46875 0.734375l0 -1.359375q1.171875 -0.5625 2.046875 -1.34375q0.890625 -0.796875 1.265625 -1.53125l0.90625 0l0 11.5z" fill-rule="nonzero"></path><path fill="#ffffff" d="m59.0 61.866142l232.0 0l0 229.13385l-232.0 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m59.0 61.866142l232.0 0l0 229.13385l-232.0 0z" fill-rule="evenodd"></path><path fill="#000000" d="m69.46875 88.78614l0 -13.359375l5.921875 0q1.78125 0 2.703125 0.359375q0.9375 0.359375 1.484375 1.28125q0.5625 0.90625 0.5625 2.015625q0 1.40625 -0.921875 2.390625q-0.921875 0.96875 -2.84375 1.234375q0.703125 0.34375 1.078125 0.671875q0.765625 0.703125 1.453125 1.765625l2.328125 3.640625l-2.21875 0l-1.765625 -2.78125q-0.78125 -1.203125 -1.28125 -1.828125q-0.5 -0.640625 -0.90625 -0.890625q-0.390625 -0.265625 -0.796875 -0.359375q-0.296875 -0.078125 -0.984375 -0.078125l-2.0
 46875 0l0 5.9375l-1.765625 0zm1.765625 -7.453125l3.796875 0q1.21875 0 1.890625 -0.25q0.6875 -0.265625 1.046875 -0.8125q0.359375 -0.546875 0.359375 -1.1875q0 -0.953125 -0.6875 -1.5625q-0.6875 -0.609375 -2.1875 -0.609375l-4.21875 0l0 4.421875zm18.097946 4.34375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm11.828842 5.765625l-3.6875 -9.671875l1.734375 0l2.078125 5.796875q0.328125 0.9375 0.625 1.9375q0.203125 -0.765625 0.609375 -1.828125l2.1406
 25 -5.90625l1.6875 0l-3.65625 9.671875l-1.53125 0zm13.265625 -3.109375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm9.125717 5.765625l0 -9.671875l1.46875 0l0 1.46875q0.5625 -1.03125 1.03125 -1.359375q0.484375 -0.328125 1.0625 -0.328125q0.828125 0 1.6875 0.53125l-0.5625 1.515625q-0.609375 -0.359375 -1.203125 -0.359375q-0.546875 0 -0.96875 0.328125q-0.421875 0.328125 -0.609375 0.890625q-0.28125 0.875 -0.28125 1.921875l0 5.0625l-1.625 0zm5.572
 052 -2.890625l1.625 -0.25q0.125 0.96875 0.75 1.5q0.625 0.515625 1.75 0.515625q1.125 0 1.671875 -0.453125q0.546875 -0.46875 0.546875 -1.09375q0 -0.546875 -0.484375 -0.875q-0.328125 -0.21875 -1.671875 -0.546875q-1.8125 -0.46875 -2.515625 -0.796875q-0.6875 -0.328125 -1.046875 -0.90625q-0.359375 -0.59375 -0.359375 -1.3125q0 -0.640625 0.296875 -1.1875q0.296875 -0.5625 0.8125 -0.921875q0.375 -0.28125 1.03125 -0.46875q0.671875 -0.203125 1.421875 -0.203125q1.140625 0 2.0 0.328125q0.859375 0.328125 1.265625 0.890625q0.421875 0.5625 0.578125 1.5l-1.609375 0.21875q-0.109375 -0.75 -0.640625 -1.171875q-0.515625 -0.421875 -1.46875 -0.421875q-1.140625 0 -1.625 0.375q-0.46875 0.375 -0.46875 0.875q0 0.3125 0.1875 0.578125q0.203125 0.265625 0.640625 0.4375q0.234375 0.09375 1.4375 0.421875q1.75 0.453125 2.4375 0.75q0.6875 0.296875 1.078125 0.859375q0.390625 0.5625 0.390625 1.40625q0 0.828125 -0.484375 1.546875q-0.46875 0.71875 -1.375 1.125q-0.90625 0.390625 -2.046875 0.390625q-1.875 0 -2.875 -0.78125q
 -0.984375 -0.78125 -1.25 -2.328125zm16.609375 -0.21875l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm14.527771 5.765625l0 -13.359375l5.046875 0q1.328125 0 2.03125 0.125q0.96875 0.171875 1.640625 0.640625q0.671875 0.453125 1.078125 1.28125q0.40625 0.828125 0.40625 1.828125q0 1.703125 -1.09375 2.890625q-1.078125 1.171875 -3.921875 1.171875l-3.421875 0l0 5.421875l-1.765625 0zm1.765625 -7.0l3.453125 0q1.71875 0 2.4375 -0.640625q0.71875 -0.640625
  0.71875 -1.796875q0 -0.84375 -0.421875 -1.4375q-0.421875 -0.59375 -1.125 -0.78125q-0.4375 -0.125 -1.640625 -0.125l-3.421875 0l0 4.78125zm10.459198 7.0l0 -9.671875l1.46875 0l0 1.46875q0.5625 -1.03125 1.03125 -1.359375q0.484375 -0.328125 1.0625 -0.328125q0.828125 0 1.6875 0.53125l-0.5625 1.515625q-0.609375 -0.359375 -1.203125 -0.359375q-0.546875 0 -0.96875 0.328125q-0.421875 0.328125 -0.609375 0.890625q-0.28125 0.875 -0.28125 1.921875l0 5.0625l-1.625 0zm5.618927 -4.84375q0 -2.6875 1.484375 -3.96875q1.25 -1.078125 3.046875 -1.078125q2.0 0 3.265625 1.3125q1.265625 1.296875 1.265625 3.609375q0 1.859375 -0.5625 2.9375q-0.5625 1.0625 -1.640625 1.65625q-1.0625 0.59375 -2.328125 0.59375q-2.03125 0 -3.28125 -1.296875q-1.25 -1.3125 -1.25 -3.765625zm1.6875 0q0 1.859375 0.796875 2.796875q0.8125 0.921875 2.046875 0.921875q1.21875 0 2.03125 -0.921875q0.8125 -0.9375 0.8125 -2.84375q0 -1.796875 -0.8125 -2.71875q-0.8125 -0.921875 -2.03125 -0.921875q-1.234375 0 -2.046875 0.921875q-0.796875 0.90625 -0
 .796875 2.765625zm8.203842 4.84375l3.53125 -5.03125l-3.265625 -4.640625l2.046875 0l1.484375 2.265625q0.421875 0.640625 0.671875 1.078125q0.40625 -0.59375 0.734375 -1.0625l1.640625 -2.28125l1.953125 0l-3.34375 4.546875l3.59375 5.125l-2.015625 0l-1.984375 -3.0l-0.515625 -0.8125l-2.546875 3.8125l-1.984375 0zm10.34375 3.71875l-0.1875 -1.53125q0.546875 0.140625 0.9375 0.140625q0.546875 0 0.875 -0.1875q0.328125 -0.171875 0.546875 -0.5q0.15625 -0.25 0.5 -1.21875q0.046875 -0.140625 0.140625 -0.40625l-3.671875 -9.6875l1.765625 0l2.015625 5.59375q0.390625 1.078125 0.703125 2.25q0.28125 -1.125 0.671875 -2.203125l2.078125 -5.640625l1.640625 0l-3.6875 9.828125q-0.59375 1.609375 -0.921875 2.203125q-0.4375 0.8125 -1.0 1.1875q-0.5625 0.375 -1.34375 0.375q-0.484375 0 -1.0625 -0.203125z" fill-rule="nonzero"></path><path fill="#ffffff" d="m3.0 11.866141l82.99213 0l0 30.362206l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt
 " d="m3.0 11.866141l82.99213 0l0 30.362206l-82.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m22.96875 29.279743l1.765625 0.453125q-0.5625 2.171875 -2.0 3.328125q-1.4375 1.140625 -3.53125 1.140625q-2.15625 0 -3.515625 -0.875q-1.34375 -0.890625 -2.0625 -2.546875q-0.703125 -1.671875 -0.703125 -3.59375q0 -2.078125 0.796875 -3.625q0.796875 -1.5625 2.265625 -2.359375q1.484375 -0.8125 3.25 -0.8125q2.0 0 3.359375 1.015625q1.375 1.015625 1.90625 2.875l-1.734375 0.40625q-0.46875 -1.453125 -1.359375 -2.109375q-0.875 -0.671875 -2.203125 -0.671875q-1.546875 0 -2.578125 0.734375q-1.03125 0.734375 -1.453125 1.984375q-0.421875 1.234375 -0.421875 2.5625q0 1.703125 0.5 2.96875q0.5 1.265625 1.546875 1.90625q1.046875 0.625 2.265625 0.625q1.484375 0 2.515625 -0.859375q1.03125 -0.859375 1.390625 -2.546875zm3.6916962 4.6875l0 -13.359375l1.640625 0l0 13.359375l-1.640625 0zm4.191696 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.64062
 5 0zm10.769821 -3.109375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm9.141342 5.765625l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65
 625 -0.75 2.515625l0 5.28125l-1.640625 0zm13.953842 -1.46875l0.234375 1.453125q-0.6875 0.140625 -1.234375 0.140625q-0.890625 0 -1.390625 -0.28125q-0.484375 -0.28125 -0.6875 -0.734375q-0.203125 -0.46875 -0.203125 -1.9375l0 -5.578125l-1.203125 0l0 -1.265625l1.203125 0l0 -2.390625l1.625 -0.984375l0 3.375l1.65625 0l0 1.265625l-1.65625 0l0 5.671875q0 0.6875 0.078125 0.890625q0.09375 0.203125 0.28125 0.328125q0.203125 0.109375 0.578125 0.109375q0.265625 0 0.71875 -0.0625zm7.323929 1.46875l-1.640625 0l0 -10.453125q-0.59375 0.5625 -1.5625 1.140625q-0.953125 0.5625 -1.71875 0.84375l0 -1.59375q1.375 -0.640625 2.40625 -1.5625q1.03125 -0.921875 1.453125 -1.78125l1.0625 0l0 13.40625z" fill-rule="nonzero"></path><path fill="#ffffff" d="m392.0 74.04724l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m392.0 74.04724l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path fill="#0
 00000" d="m402.23438 100.96724l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65625 -0.75 2.515625l0 5.28125l-1.640625 0zm10.375732 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm4.535431 0l0 -8.40625l-1.453125 0l0 -1.265625l1.453125 0l0 -1.03125q0 -0.96875 0.171875 -1.453125q0.234375 -0.640625 0.828125 -1.03125q0.59375 -0.390625 1.671875 -0.390625q0.6875 0 1.53125 0.15625l-0.25 1.4375q-0.5 -0.09375 -0.953125 -0.09375q-0.75 0 -1.0625 0.328125q-0.3125 0.3125 -0.3125 1.1875l0 0.890625l1.890625 0l0 1.265625l-1.890625 0l0 8.40625l-1.625 0zm4.792694 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -
 9.671875l1.640625 0l0 9.671875l-1.640625 0zm3.691681 -6.59375q0 -2.359375 0.484375 -3.796875q0.484375 -1.453125 1.4375 -2.234375q0.96875 -0.78125 2.421875 -0.78125q1.078125 0 1.890625 0.4375q0.8125 0.421875 1.328125 1.25q0.53125 0.8125 0.828125 1.984375q0.3125 1.15625 0.3125 3.140625q0 2.359375 -0.484375 3.8125q-0.484375 1.4375 -1.453125 2.234375q-0.953125 0.78125 -2.421875 0.78125q-1.921875 0 -3.03125 -1.390625q-1.3125 -1.671875 -1.3125 -5.4375zm1.671875 0q0 3.296875 0.765625 4.390625q0.78125 1.078125 1.90625 1.078125q1.140625 0 1.90625 -1.09375q0.765625 -1.09375 0.765625 -4.375q0 -3.296875 -0.765625 -4.375q-0.765625 -1.078125 -1.921875 -1.078125q-1.125 0 -1.796875 0.953125q-0.859375 1.21875 -0.859375 4.5z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m76.0 104.92913l202.99213 0l0 34.3937l-202.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m76.0 104.92913l202.99213 0l0 34.3937l-202.99213 0z" fill
 -rule="evenodd"></path><path fill="#000000" d="m86.0625 127.985985l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.8437
 5l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.7265625 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.7031174 -2.359375 0.7031174q-1.828125 0 -2.890625 -1.1249924q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm6.8984375 4.953125l3.03125 -4.3125l-2.8125 -3.984375l1.765625 0l1.265625 1.9375q0.359375 0.5625 0.578125 0.9375q0.34375 -0.515625 0.640625 -0.921875l1.390625 -1.953125l1.6875 0l-2.875 3.90625l3.09375 4.390625l-1.734
 375 0l-1.703125 -2.578125l-0.453125 -0.703125l-2.171875 3.28125l-1.703125 0zm14.34375 -1.03125q-0.78125 0.671875 -1.5 0.953125q-0.71875 0.26561737 -1.546875 0.26561737q-1.375 0 -2.109375 -0.6718674q-0.734375 -0.671875 -0.734375 -1.703125q0 -0.609375 0.28125 -1.109375q0.28125 -0.515625 0.71875 -0.8125q0.453125 -0.3125 1.015625 -0.46875q0.421875 -0.109375 1.25 -0.203125q1.703125 -0.203125 2.515625 -0.484375q0 -0.296875 0 -0.375q0 -0.859375 -0.390625 -1.203125q-0.546875 -0.484375 -1.609375 -0.484375q-0.984375 0 -1.46875 0.359375q-0.46875 0.34375 -0.6875 1.21875l-1.375 -0.1875q0.1875 -0.875 0.609375 -1.421875q0.4375 -0.546875 1.25 -0.828125q0.8125 -0.296875 1.875 -0.296875q1.0625 0 1.71875 0.25q0.671875 0.25 0.984375 0.625q0.3125 0.375 0.4375 0.953125q0.078125 0.359375 0.078125 1.296875l0 1.875q0 1.96875 0.078125 2.484375q0.09375 0.515625 0.359375 1.0l-1.46875 0q-0.21875 -0.4375 -0.28125 -1.03125zm-0.109375 -3.140625q-0.765625 0.3125 -2.296875 0.53125q-0.875 0.125 -1.234375 0.28125q-0.3
 59375 0.15625 -0.5625 0.46875q-0.1875 0.296875 -0.1875 0.65625q0 0.5625 0.421875 0.9375q0.4375 0.375 1.25 0.375q0.8125 0 1.4375 -0.34375q0.640625 -0.359375 0.9375 -0.984375q0.234375 -0.46875 0.234375 -1.40625l0 -0.515625zm3.6015625 4.171875l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm13.328125 3.1874924l0 -11.484367l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q
 0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.5468674 -1.828125 0.5468674q-0.703125 0 -1.265625 -0.29686737q-0.546875 -0.296875 -0.90625 -0.75l0 4.0468674l-1.40625 0zm1.265625 -7.2968674q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.6015625 4.109375l0 -11.453125l1.40625 0l0 11.453125l-1.40625 0zm9.2578125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.7031174 -2.359375 0.7031174q-1.828125 0 -2.890625 -1.1249924q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.
 375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.4609375 -3.046875l1.390625 0.1875q-0.234375 1.421875 -1.171875 2.234375q-0.921875 0.8124924 -2.28125 0.8124924q-1.703125 0 -2.75 -1.1093674q-1.03125 -1.125 -1.03125 -3.203125q0 -1.34375 0.4375 -2.34375q0.453125 -1.015625 1.359375 -1.515625q0.921875 -0.5 1.984375 -0.5q1.359375 0 2.21875 0.6875q0.859375 0.671875 1.09375 1.9375l-1.359375 0.203125q-0.203125 -0.828125 -0.703125 -1.25q-0.484375 -0.421875 -1.1875 -0.421875q-1.0625 0 -1.734375 0.765625q-0.65625 0.75 -0.65625 2.40625q0 1.671875 0.640625 2.4375q0.640625 0.75 1.671875 0.75q0.828125 0 1.375 -0.5q0.5625 -0.515625 0.703125 -1.578125zm2.0625 -1.109375q0 -2.296875 1.28125 -3.40625q1.0
 78125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.49999237 -2.0 0.49999237q-1.734375 0 -2.8125 -1.1093674q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.9765625 4.15625l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.859375l
 0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm13.71875 -6.703125l0 -1.59375l1.59375 0l0 1.59375l-1.59375 0zm0 6.703125l0 -1.609375l1.59375 0l0 1.609375l-1.59375 0zm8.1640625 0l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0
 .21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.0468674 -2.765625 1.0468674q-1.5 0 -2.5 -0.8906174q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m278.99213 122.125984l65.502625 0l0 0.06299591l65.520996 0" fill-rule="evenodd"></path><path stroke="#ff0000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" stroke-dasharray="4.0,3.0" d="m278.99213 122.125984l65.502625 0l0 0.06299591l59.520996 0" fill-rule="evenodd"></
 path><path fill="#ff0000" stroke="#ff0000" stroke-width="1.0" stroke-linecap="butt" d="m404.01575 123.840706l4.538086 -1.6517258l-4.538086 -1.6517334z" fill-rule="evenodd"></path><path fill="#cfe2f3" d="m410.0 109.51181l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m410.0 109.51181l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m421.82812 121.767235q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 0.9999924 -2.734375 0.9999924q-1.6875 0 -2.734375 -0.9999924q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -
 0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.398
 4375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.0468674 -2.765625 1.0468674q-1.5 0 -2.5 -0.8906174q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m410.0 141.58267l57.00787
 4 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m410.0 141.58267l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m421.82812 153.8381q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0
 .515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375
  1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015
 625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm12.7734375 3.3125l-1.40625 0l0 -8.96875q-0.515625 0.484375 -1.34375 0.96875q-0.8125 0.484375 -1.46875 0.734375l0 -1.359375q1.171875 -0.5625 2.046875 -1.34375q0.890625 -0.796875 1.265625 -1.53125l0.90625 0l0 11.5z" fill-rule="nonzero"></path><path fill="#ffffff" d="m470.0 135.81102l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m470.0 135.81102l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m480.23438 162.73102l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -
 0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65625 -0.75 2.515625l0 5.28125l-1.640625 0zm10.375732 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm4.535431 0l0 -8.40625l-1.453125 0l0 -1.265625l1.453125 0l0 -1.03125q0 -0.96875 0.171875 -1.453125q0.234375 -0.640625 0.828125 -1.03125q0.59375 -0.390625 1.671875 -0.390625q0.6875 0 1.53125 0.15625l-0.25 1.4375q-0.5 -0.09375 -0.953125 -0.09375q-0.75 0 -1.0625 0.328125q-0.3125 0.3125 -0.3125 1.1875l0 0.890625l1.890625 0l0 1.265625l-1.890625 0l0 8.40625l-1.625 0zm4.792694 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm9.863556 0l-1.640625 0l0 -10.453125q-0.59375 0.5625 -1.5625 1.140625q-0.953125 0.5625 -1.71875 0.84375l0 -1.59375q1.375 -0.640625 2.40625 -1.5625q1.03125 -0.921875 1.453125 -1.78125l1.0625 0l0 13.40625z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m484.10278 172.34645l57.007874 0l0 25.22
 8348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m484.10278 172.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m495.9309 184.60188q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625
 q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.
 234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.046875 -2.765625 1.046875q-1.5 0 -2.5 -0.890625q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m484.10278 203.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m484.10278 203.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#00000
 0" d="m495.9309 215.60188q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q
 -0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 
 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53

<TRUNCATED>

[2/5] nifi git commit: NIFI-4932: Enable S2S work behind a Reverse Proxy Adding S2S endpoint Reverse Proxy mapping capability. Added license header to SVG files. Incorporated review comments. Use regex to check property key processing. Catch AttributeExp

Posted by mc...@apache.org.
http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-docs/src/main/asciidoc/images/s2s-rproxy-servername.svg
----------------------------------------------------------------------
diff --git a/nifi-docs/src/main/asciidoc/images/s2s-rproxy-servername.svg b/nifi-docs/src/main/asciidoc/images/s2s-rproxy-servername.svg
new file mode 100644
index 0000000..2f68e08
--- /dev/null
+++ b/nifi-docs/src/main/asciidoc/images/s2s-rproxy-servername.svg
@@ -0,0 +1,17 @@
+<?xml version="1.0" standalone="yes"?>
+<!--
+  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 version="1.1" viewBox="0.0 0.0 800.0 450.0" fill="none" stroke="none" stroke-linecap="square" stroke-miterlimit="10" xmlns="http://www.w3.org/2000/svg" xmlns:xlink="http://www.w3.org/1999/xlink"><clipPath id="p.0"><path d="m0 0l800.0 0l0 600.0l-800.0 0l0 -600.0z" clip-rule="nonzero"></path></clipPath><g clip-path="url(#p.0)"><path fill="#000000" fill-opacity="0.0" d="m0 0l800.0 0l0 600.0l-800.0 0z" fill-rule="evenodd"></path><path fill="#000000" fill-opacity="0.0" d="m85.99213 13.036745l214.99213 0l0 34.3937l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m96.05463 37.396748l0 -11.453127l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.8593769l0 5.265625l-1.40625 0l0 -5.265625q0 -1.0468769 -0.453125 -1.5312519q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.5156269l0 4.546875l-1.4
 0625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765627l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.843752q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765627l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.843752q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484377l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.5
 46875q0.421875 0.984375 0.421875 2.171877q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.5937519 -0.65625 -2.375002q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.437502zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5000019 -0.296875 -1.1093769q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.2812
 5q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.3750019q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 -4.218752l0 -1.59375l1.59375 0l0 1.59375l-1.59375 0zm0 6.703127l0 -1.609375l1.59375 0l0 1.609375l-1.59375 0zm2.9921875 0.203125l3.328125 -11.859377l1.125 0l-3.3125 11.859377l-1.140625 0zm4.4453125 0l3.328125 -11.859377l1.125 0l-3.3125 11.859377l-1.140625 0zm5.5078125 -0.203125l0 -8.296877l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0
 .4375 1.046875q0.078125 0.390625 0.078125 1.3593769l0 5.109375l-1.40625 0l0 -5.046875q0 -0.8593769 -0.171875 -1.2812519q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.156252l0 4.53125l-1.40625 0zm8.8984375 -9.843752l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843752l0 -8.296877l1.40625 0l0 8.296877l-1.40625 0zm3.8828125 0l0 -7.203127l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203127l-1.40625 0zm4.1171875 -9.843752l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843752l0 -8.296877l1.40625 0l0 8.296877l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.7265625 -2.671875l1.453125 0.17187
 5q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.093752 1.078125 -3.250002q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171877q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.0468769 -0.53125 -1.5625019q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.7187519zm6.8984375 4.953125l3.03125 -4.3125l-2.8125 -3.984377l1.765625 0l1.265625 1.9375q0.359375 0.5625 0.578125 0.9375q0.34375 -0.515625 0.640625 -0.921875l1.390625 -1.953125l1.6875 0l-2.875 3.906252l3.09375 4.390625l-1.734375 0l-1.703125 -2.578125l-0.453125 -0.703125l-2.171875 3.28125l-1.703125 0zm14.34375 -1.03125q-0.78125 0.671875 -1.5 0.953125q-0.71875 0.265625 -1.546875 0.265625q-1.375 0 -2.109375 -0.67187
 5q-0.734375 -0.671875 -0.734375 -1.703125q0 -0.609375 0.28125 -1.109375q0.28125 -0.515625 0.71875 -0.8125q0.453125 -0.3125 1.015625 -0.46875q0.421875 -0.109375 1.25 -0.203125q1.703125 -0.203125 2.515625 -0.484375q0 -0.2968769 0 -0.3750019q0 -0.859375 -0.390625 -1.203125q-0.546875 -0.484375 -1.609375 -0.484375q-0.984375 0 -1.46875 0.359375q-0.46875 0.34375 -0.6875 1.21875l-1.375 -0.1875q0.1875 -0.875 0.609375 -1.421875q0.4375 -0.546875 1.25 -0.828125q0.8125 -0.296875 1.875 -0.296875q1.0625 0 1.71875 0.25q0.671875 0.25 0.984375 0.625q0.3125 0.375 0.4375 0.953125q0.078125 0.359375 0.078125 1.2968769l0 1.875q0 1.96875 0.078125 2.484375q0.09375 0.515625 0.359375 1.0l-1.46875 0q-0.21875 -0.4375 -0.28125 -1.03125zm-0.109375 -3.140625q-0.765625 0.3125 -2.296875 0.53125q-0.875 0.125 -1.234375 0.28125q-0.359375 0.15625 -0.5625 0.46875q-0.1875 0.296875 -0.1875 0.65625q0 0.5625 0.421875 0.9375q0.4375 0.375 1.25 0.375q0.8125 0 1.4375 -0.34375q0.640625 -0.359375 0.9375 -0.984375q0.234375 -0.46875
  0.234375 -1.40625l0 -0.515625zm3.6015625 4.171875l0 -8.296877l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703127l-1.390625 0l0 -5.234375q0 -0.8437519 -0.140625 -1.2031269q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.8593769l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375019 -0.34375 -1.4062519q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.7187519l0 4.3125l-1.40625 0zm13.328125 3.1875l0 -11.484377l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171877q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.32812
 5 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.5937519 -0.65625 -2.375002q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.437502zm7.6015625 4.109375l0 -11.453127l1.40625 0l0 11.453127l-1.40625 0zm9.2578125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.093752 1.078125 -3.250002q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171877q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.0468769 -0.53125 -1.5
 625019q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.7187519zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.4609375 -3.046875l1.390625 0.1875q-0.234375 1.421875 -1.171875 2.234375q-0.921875 0.8125 -2.28125 0.8125q-1.703125 0 -2.75 -1.109375q-1.03125 -1.125 -1.03125 -3.203125q0 -1.3437519 0.4375 -2.343752q0.453125 -1.015625 1.359375 -1.515625q0.921875 -0.5 1.984375 -0.5q1.359375 0 2.21875 0.6875q0.859375 0.671875 1.09375 1.9375l-1.359375 0.203125q-0.203125 -0.828125 -0.703125 -1.25q-0.484375 -0.421875 -1.1875 -0.421875q-1.0625 0 -1.734375 0.765625q-0.65625 0.75 -0.65625 2.406252q0 1.671875 0.640625 2.4375q0.640625 0.75 1.671875 0.75q0.828125 0 1.375 -0.5q0.5625 -0.515625 0.703125 -1.578125zm2.0625 -1.109375q0 -2.296877 1.28125 -3.406252q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.093752q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.
 5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.5312519 -0.703125 -2.328127q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375002zm7.9765625 4.15625l0 -8.296877l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703127l-1.390625 0l0 -5.234375q0 -0.8437519 -0.140625 -1.2031269q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.8593769l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375019 -0.34375 -1.4062519q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.2187
 5 1.7187519l0 4.3125l-1.40625 0z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m476.09448 69.94226l214.99213 0l0 58.04724l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m486.15698 94.30226l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.82
 8125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm5.5859375 0l-2.546875 -8.296875l1.453125 0l1.328125 4.78125l0.484375 1.78125q0.03125 -0.125 0.4375 -1.703125l1.3125 -4.859375l1.453125 0l1.234375 4.8125l0.421875 1.578125l0.46875 -1.59375l1.421875 -4.796875l1.375 0l-2.59375 8.296875l-1.46875 0l-1.3125 -4.96875l-0.328125 -1.421875l-1.671875 6.390625l-1.46875 0zm15.6953125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.7031
 25 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm9.1328125 4.953125l-1.3125 0l0 -11.453125l1.40625 0l0 4.078125q0.890625 -1.109375 2.28125 -1.109375q0.765625 0 1.4375 0.3125q0.6875 0.296875 1.125 0.859375q0.453125 0.5625 0.703125 1.359375q0.25 0.78125 0.25 1.671875q0 2.140625 -1.0625 3.3125q-1.046875 1.15625 -2.53125 1.15625q-1.46875 0 -2.296875 -1.234375l0 1.046875zm-0.015625 -4.21875q0 1.5 0.40625 2.15625q0.65625 1.09375 1.796875 1.09375q0.921875 0 1.59375 -0.796875q0.671875 -0.8125 0.671875 -2.390625q0 -1.625 -0.65625 -2.390625q-0.640625 -0.78125 -1.546875 -0.78125q-0.921875 0 -1.59375 0.796875q-0.671875 0.796875 -0.671875 2.3125zm8.0078125 4.21875l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.1406
 25q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-
 1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 
 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 2.484375l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.14062
 5 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm8.3671875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.4140625 1.671875l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0
 .4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm11.625 1.21875l0.203125 1.25q-0.59375 0.12
 5 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm1.953125 3.265625l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.29687
 5l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.1640625 -5.65625q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640
 625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375z" fill-rule="nonzero"></path><path fill="#000000" d="m486.15698 113.30226l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.2
 03125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0390625 0l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm11.015625 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.18
 75 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm7.8359375 4.953125l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm12.796875 -4
 .15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm11.0390625 2.890625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm7.0546875 -1.40625l1.453125 0.171875q-0.
 34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.5546875 0l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 
 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm13.0703125 4.109375l0 -1.21875q-0.96875 1.40625 -2.640625 1.40625q-0.734375 0 -1.375 -0.28125q-0.625 -0.28125 -0.9375 -0.703125q-0.3125 -0.
 4375 -0.4375 -1.046875q-0.078125 -0.421875 -0.078125 -1.3125l0 -5.140625l1.40625 0l0 4.59375q0 1.109375 0.078125 1.484375q0.140625 0.5625 0.5625 0.875q0.4375 0.3125 1.0625 0.3125q0.640625 0 1.1875 -0.3125q0.5625 -0.328125 0.78125 -0.890625q0.234375 -0.5625 0.234375 -1.625l0 -4.4375l1.40625 0l0 8.296875l-1.25 0zm6.5234375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.7734375 1.265625l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625
 q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm8.3671875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.4140625 1.671875l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0.46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625
  -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm11.625 1.21875l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -
 1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm1.953125 3.265625l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.8
 90625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.1640625 -5.65625q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546
 875 0.828125q-0.734375 1.046875 -0.734375 3.859375z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m392.0 4.553806l214.99213 0l0 58.047245l-214.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m402.0625 28.913807l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375
  -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm5.5859375 0l-2.546875 -8.296875l1.453125 0l1.328125 4.78125l0.484375 1.78125q0.03125 -0.125 0.4375 -1.703125l1.3125 -4.859375l1.453125 0l1.234375 4.8125l0.421875 1.578125l0.46875 -1.59375l1.421875 -4.796875l1.375 0l-2.59375 8.296875l-1.46875 0l-1.3125 -4.96875l-0.328125 -1.421875l-1.671875 6.390625l-1.46875 0zm15.6953125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625
  2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm9.1328125 4.953125l-1.3125 0l0 -11.453125l1.40625 0l0 4.078125q0.890625 -1.109375 2.28125 -1.109375q0.765625 0 1.4375 0.3125q0.6875 0.296875 1.125 0.859375q0.453125 0.5625 0.703125 1.359375q0.25 0.78125 0.25 1.671875q0 2.140625 -1.0625 3.3125q-1.046875 1.15625 -2.53125 1.15625q-1.46875 0 -2.296875 -1.234375l0 1.046875zm-0.015625 -4.21875q0 1.5 0.40625 2.15625q0.65625 1.09375 1.796875 1.09375q0.921875 0 1.59375 -0.796875q0.671875 -0.8125 0.671875 -2.390625q0 -1.625 -0.65625 -2.390625q-0.640625 -0.78125 -1.546875 -0.78125q-0.921875 0 -1.59375 0.796875q-0.671875 0.796875 -0.671875 2.3125zm8.0078125 4.21875l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 0l0 -11.453125l1.40625 0l0 4.109375q0.984375 -1.140625 
 2.484375 -1.140625q0.921875 0 1.59375 0.359375q0.6875 0.359375 0.96875 1.0q0.296875 0.640625 0.296875 1.859375l0 5.265625l-1.40625 0l0 -5.265625q0 -1.046875 -0.453125 -1.53125q-0.453125 -0.484375 -1.296875 -0.484375q-0.625 0 -1.171875 0.328125q-0.546875 0.328125 -0.78125 0.890625q-0.234375 0.546875 -0.234375 1.515625l0 4.546875l-1.40625 0zm11.9609375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm4.4453125 0l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.406
 25 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.3828125 4.453125l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.0703125 1.625l1.390625 -0.21875q0.109375 0.84375 0.640625 1.296875q0.546875 0.4375 1.5 0.4375q0.96875 0 1.4375 -0.390625q0.46875 -0.40625 0.46875 -0.9375q0 -0
 .46875 -0.40625 -0.75q-0.296875 -0.1875 -1.4375 -0.46875q-1.546875 -0.390625 -2.15625 -0.671875q-0.59375 -0.296875 -0.90625 -0.796875q-0.296875 -0.5 -0.296875 -1.109375q0 -0.5625 0.25 -1.03125q0.25 -0.46875 0.6875 -0.78125q0.328125 -0.25 0.890625 -0.40625q0.578125 -0.171875 1.21875 -0.171875q0.984375 0 1.71875 0.28125q0.734375 0.28125 1.078125 0.765625q0.359375 0.46875 0.5 1.28125l-1.375 0.1875q-0.09375 -0.640625 -0.546875 -1.0q-0.453125 -0.359375 -1.265625 -0.359375q-0.96875 0 -1.390625 0.328125q-0.40625 0.3125 -0.40625 0.734375q0 0.28125 0.171875 0.5q0.171875 0.21875 0.53125 0.375q0.21875 0.078125 1.25 0.359375q1.484375 0.390625 2.078125 0.65625q0.59375 0.25 0.921875 0.734375q0.34375 0.484375 0.34375 1.203125q0 0.703125 -0.421875 1.328125q-0.40625 0.609375 -1.1875 0.953125q-0.765625 0.34375 -1.734375 0.34375q-1.625 0 -2.46875 -0.671875q-0.84375 -0.671875 -1.078125 -2.0zm8.953125 2.484375l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 3.1875l0 -11.484375l1.28125 0l0 1.078
 125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.1015625 -0.046875q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.7968
 75q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.9609375 4.15625l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm8.40625 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-
 7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm3.71875 -2.953125q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q
 0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.5
 15625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.046875 -2.765625 1.046875q-1.5 0 -2.5 -0.890625q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#000000" d="m402.0625 47.913807l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.843
 75l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.8828125 0l0 -7.203125l-1.234375 0l0 -1.09375l1.234375 0l0 -0.890625q0 -0.828125 0.15625 -1.234375q0.203125 -0.546875 0.703125 -0.890625q0.515625 -0.34375 1.4375 -0.34375q0.59375 0 1.3125 0.140625l-0.203125 1.234375q-0.4375 -0.078125 -0.828125 -0.078125q-0.640625 0 -0.90625 0.28125q-0.265625 0.265625 -0.265625 1.015625l0 0.765625l1.609375 0l0 1.09375l-1.609375 0l0 7.203125l-1.40625 0zm4.1171875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l-1.40625 0zm3.9453125 0l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0390625 0l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm11.015625 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125
  1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm7.8359375 4.953125l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125
  0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm12.796875 -4.15625q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm11.0390625 2.890625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.0937
 5l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm7.0546875 -1.40625l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.703125 -2.359375 0.703125q-1.828125 0 -2.890625 -1.125q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 -9.84375l0 -1.609375l1.40625 0l0 1.609375l-1.40625 0zm0 9.84375l0 -8.296875l1.40625 0l0 8.296875l
 -1.40625 0zm3.5546875 0l0 -8.296875l1.265625 0l0 1.171875q0.90625 -1.359375 2.640625 -1.359375q0.75 0 1.375 0.265625q0.625 0.265625 0.9375 0.703125q0.3125 0.4375 0.4375 1.046875q0.078125 0.390625 0.078125 1.359375l0 5.109375l-1.40625 0l0 -5.046875q0 -0.859375 -0.171875 -1.28125q-0.15625 -0.4375 -0.578125 -0.6875q-0.40625 -0.25 -0.96875 -0.25q-0.90625 0 -1.5625 0.578125q-0.640625 0.5625 -0.640625 2.15625l0 4.53125l-1.40625 0zm8.8984375 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.
 65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm13.0703125 4.109375l0 -1.21875q-0.96875 1.40625 -2.640625 1.40625q-0.734375 0 -1.375 -0.28125q-0.625 -0.28125 -0.9375 -0.703125q-0.3125 -0.4375 -0.4375 -1.046875q-0.078125 -0.421875 -0.078125 -1.3125l0 -5.140625l1.40625 0l0 4.59375q0 1.109375 0.078125 1.484375q0.140625 0.5625 0.5625 0.875q0.4375 0.3125 1.0625 0.3125q0.640625 0 1.1875 -0.3125q0.5625 -0.328125 0.78125 -0.890625q0.234375 -0.5625 0.234375 -1.625l0 -4.4375l1.40625 0l0 8.296875l-1.25 0zm6.5234375 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0 0.609375 -0.0625zm1.7734375 1.265625l0 -1.6
 09375l1.609375 0l0 1.609375l-1.609375 0zm4.0546875 3.1875l0 -11.484375l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.546875 -1.828125 0.546875q-0.703125 0 -1.265625 -0.296875q-0.546875 -0.296875 -0.90625 -0.75l0 4.046875l-1.40625 0zm1.265625 -7.296875q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.1015625 -0.046875q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.5 -2.0 0.5q-1.734375 0 -2.8125 -1.109375q-1.078125 -1.125
  -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.9609375 4.15625l0 -8.296875l1.265625 0l0 1.25q0.484375 -0.875 0.890625 -1.15625q0.40625 -0.28125 0.90625 -0.28125q0.703125 0 1.4375 0.453125l-0.484375 1.296875q-0.515625 -0.296875 -1.03125 -0.296875q-0.453125 0 -0.828125 0.28125q-0.359375 0.265625 -0.515625 0.765625q-0.234375 0.75 -0.234375 1.640625l0 4.34375l-1.40625 0zm8.40625 -1.265625l0.203125 1.25q-0.59375 0.125 -1.0625 0.125q-0.765625 0 -1.1875 -0.234375q-0.421875 -0.25 -0.59375 -0.640625q-0.171875 -0.40625 -0.171875 -1.671875l0 -4.765625l-1.03125 0l0 -1.09375l1.03125 0l0 -2.0625l1.40625 -0.84375l0 2.90625l1.40625 0l0 1.09375l-1.40625 0l0 4.84375q0 0.609375 0.0625 0.78125q0.078125 0.171875 0.25 0.28125q0.171875 0.09375 0.484375 0.09375q0.234375 0
  0.609375 -0.0625zm8.7734375 -5.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm0 3.46875l-7.5625 0l0 -1.3125l7.5625 0l0 1.3125zm3.71875 -2.953125q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.2
 96875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.8
 90625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm12.773
 4375 3.3125l-1.40625 0l0 -8.96875q-0.515625 0.484375 -1.34375 0.96875q-0.8125 0.484375 -1.46875 0.734375l0 -1.359375q1.171875 -0.5625 2.046875 -1.34375q0.890625 -0.796875 1.265625 -1.53125l0.90625 0l0 11.5z" fill-rule="nonzero"></path><path fill="#ffffff" d="m59.0 61.866142l232.0 0l0 229.13385l-232.0 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m59.0 61.866142l232.0 0l0 229.13385l-232.0 0z" fill-rule="evenodd"></path><path fill="#000000" d="m69.46875 88.78614l0 -13.359375l5.921875 0q1.78125 0 2.703125 0.359375q0.9375 0.359375 1.484375 1.28125q0.5625 0.90625 0.5625 2.015625q0 1.40625 -0.921875 2.390625q-0.921875 0.96875 -2.84375 1.234375q0.703125 0.34375 1.078125 0.671875q0.765625 0.703125 1.453125 1.765625l2.328125 3.640625l-2.21875 0l-1.765625 -2.78125q-0.78125 -1.203125 -1.28125 -1.828125q-0.5 -0.640625 -0.90625 -0.890625q-0.390625 -0.265625 -0.796875 -0.359375q-0.296875 -0.078125 -0.984375 -0.078125l-2.0
 46875 0l0 5.9375l-1.765625 0zm1.765625 -7.453125l3.796875 0q1.21875 0 1.890625 -0.25q0.6875 -0.265625 1.046875 -0.8125q0.359375 -0.546875 0.359375 -1.1875q0 -0.953125 -0.6875 -1.5625q-0.6875 -0.609375 -2.1875 -0.609375l-4.21875 0l0 4.421875zm18.097946 4.34375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm11.828842 5.765625l-3.6875 -9.671875l1.734375 0l2.078125 5.796875q0.328125 0.9375 0.625 1.9375q0.203125 -0.765625 0.609375 -1.828125l2.1406
 25 -5.90625l1.6875 0l-3.65625 9.671875l-1.53125 0zm13.265625 -3.109375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm9.125717 5.765625l0 -9.671875l1.46875 0l0 1.46875q0.5625 -1.03125 1.03125 -1.359375q0.484375 -0.328125 1.0625 -0.328125q0.828125 0 1.6875 0.53125l-0.5625 1.515625q-0.609375 -0.359375 -1.203125 -0.359375q-0.546875 0 -0.96875 0.328125q-0.421875 0.328125 -0.609375 0.890625q-0.28125 0.875 -0.28125 1.921875l0 5.0625l-1.625 0zm5.572
 052 -2.890625l1.625 -0.25q0.125 0.96875 0.75 1.5q0.625 0.515625 1.75 0.515625q1.125 0 1.671875 -0.453125q0.546875 -0.46875 0.546875 -1.09375q0 -0.546875 -0.484375 -0.875q-0.328125 -0.21875 -1.671875 -0.546875q-1.8125 -0.46875 -2.515625 -0.796875q-0.6875 -0.328125 -1.046875 -0.90625q-0.359375 -0.59375 -0.359375 -1.3125q0 -0.640625 0.296875 -1.1875q0.296875 -0.5625 0.8125 -0.921875q0.375 -0.28125 1.03125 -0.46875q0.671875 -0.203125 1.421875 -0.203125q1.140625 0 2.0 0.328125q0.859375 0.328125 1.265625 0.890625q0.421875 0.5625 0.578125 1.5l-1.609375 0.21875q-0.109375 -0.75 -0.640625 -1.171875q-0.515625 -0.421875 -1.46875 -0.421875q-1.140625 0 -1.625 0.375q-0.46875 0.375 -0.46875 0.875q0 0.3125 0.1875 0.578125q0.203125 0.265625 0.640625 0.4375q0.234375 0.09375 1.4375 0.421875q1.75 0.453125 2.4375 0.75q0.6875 0.296875 1.078125 0.859375q0.390625 0.5625 0.390625 1.40625q0 0.828125 -0.484375 1.546875q-0.46875 0.71875 -1.375 1.125q-0.90625 0.390625 -2.046875 0.390625q-1.875 0 -2.875 -0.78125q
 -0.984375 -0.78125 -1.25 -2.328125zm16.609375 -0.21875l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm14.527771 5.765625l0 -13.359375l5.046875 0q1.328125 0 2.03125 0.125q0.96875 0.171875 1.640625 0.640625q0.671875 0.453125 1.078125 1.28125q0.40625 0.828125 0.40625 1.828125q0 1.703125 -1.09375 2.890625q-1.078125 1.171875 -3.921875 1.171875l-3.421875 0l0 5.421875l-1.765625 0zm1.765625 -7.0l3.453125 0q1.71875 0 2.4375 -0.640625q0.71875 -0.640625
  0.71875 -1.796875q0 -0.84375 -0.421875 -1.4375q-0.421875 -0.59375 -1.125 -0.78125q-0.4375 -0.125 -1.640625 -0.125l-3.421875 0l0 4.78125zm10.459198 7.0l0 -9.671875l1.46875 0l0 1.46875q0.5625 -1.03125 1.03125 -1.359375q0.484375 -0.328125 1.0625 -0.328125q0.828125 0 1.6875 0.53125l-0.5625 1.515625q-0.609375 -0.359375 -1.203125 -0.359375q-0.546875 0 -0.96875 0.328125q-0.421875 0.328125 -0.609375 0.890625q-0.28125 0.875 -0.28125 1.921875l0 5.0625l-1.625 0zm5.618927 -4.84375q0 -2.6875 1.484375 -3.96875q1.25 -1.078125 3.046875 -1.078125q2.0 0 3.265625 1.3125q1.265625 1.296875 1.265625 3.609375q0 1.859375 -0.5625 2.9375q-0.5625 1.0625 -1.640625 1.65625q-1.0625 0.59375 -2.328125 0.59375q-2.03125 0 -3.28125 -1.296875q-1.25 -1.3125 -1.25 -3.765625zm1.6875 0q0 1.859375 0.796875 2.796875q0.8125 0.921875 2.046875 0.921875q1.21875 0 2.03125 -0.921875q0.8125 -0.9375 0.8125 -2.84375q0 -1.796875 -0.8125 -2.71875q-0.8125 -0.921875 -2.03125 -0.921875q-1.234375 0 -2.046875 0.921875q-0.796875 0.90625 -0
 .796875 2.765625zm8.203842 4.84375l3.53125 -5.03125l-3.265625 -4.640625l2.046875 0l1.484375 2.265625q0.421875 0.640625 0.671875 1.078125q0.40625 -0.59375 0.734375 -1.0625l1.640625 -2.28125l1.953125 0l-3.34375 4.546875l3.59375 5.125l-2.015625 0l-1.984375 -3.0l-0.515625 -0.8125l-2.546875 3.8125l-1.984375 0zm10.34375 3.71875l-0.1875 -1.53125q0.546875 0.140625 0.9375 0.140625q0.546875 0 0.875 -0.1875q0.328125 -0.171875 0.546875 -0.5q0.15625 -0.25 0.5 -1.21875q0.046875 -0.140625 0.140625 -0.40625l-3.671875 -9.6875l1.765625 0l2.015625 5.59375q0.390625 1.078125 0.703125 2.25q0.28125 -1.125 0.671875 -2.203125l2.078125 -5.640625l1.640625 0l-3.6875 9.828125q-0.59375 1.609375 -0.921875 2.203125q-0.4375 0.8125 -1.0 1.1875q-0.5625 0.375 -1.34375 0.375q-0.484375 0 -1.0625 -0.203125z" fill-rule="nonzero"></path><path fill="#ffffff" d="m3.0 11.866141l82.99213 0l0 30.362206l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt
 " d="m3.0 11.866141l82.99213 0l0 30.362206l-82.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m22.96875 29.279743l1.765625 0.453125q-0.5625 2.171875 -2.0 3.328125q-1.4375 1.140625 -3.53125 1.140625q-2.15625 0 -3.515625 -0.875q-1.34375 -0.890625 -2.0625 -2.546875q-0.703125 -1.671875 -0.703125 -3.59375q0 -2.078125 0.796875 -3.625q0.796875 -1.5625 2.265625 -2.359375q1.484375 -0.8125 3.25 -0.8125q2.0 0 3.359375 1.015625q1.375 1.015625 1.90625 2.875l-1.734375 0.40625q-0.46875 -1.453125 -1.359375 -2.109375q-0.875 -0.671875 -2.203125 -0.671875q-1.546875 0 -2.578125 0.734375q-1.03125 0.734375 -1.453125 1.984375q-0.421875 1.234375 -0.421875 2.5625q0 1.703125 0.5 2.96875q0.5 1.265625 1.546875 1.90625q1.046875 0.625 2.265625 0.625q1.484375 0 2.515625 -0.859375q1.03125 -0.859375 1.390625 -2.546875zm3.6916962 4.6875l0 -13.359375l1.640625 0l0 13.359375l-1.640625 0zm4.191696 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.64062
 5 0zm10.769821 -3.109375l1.6875 0.203125q-0.40625 1.484375 -1.484375 2.3125q-1.078125 0.8125 -2.765625 0.8125q-2.125 0 -3.375 -1.296875q-1.234375 -1.3125 -1.234375 -3.671875q0 -2.453125 1.25 -3.796875q1.265625 -1.34375 3.265625 -1.34375q1.9375 0 3.15625 1.328125q1.234375 1.3125 1.234375 3.703125q0 0.15625 0 0.4375l-7.21875 0q0.09375 1.59375 0.90625 2.453125q0.8125 0.84375 2.015625 0.84375q0.90625 0 1.546875 -0.46875q0.640625 -0.484375 1.015625 -1.515625zm-5.390625 -2.65625l5.40625 0q-0.109375 -1.21875 -0.625 -1.828125q-0.78125 -0.953125 -2.03125 -0.953125q-1.125 0 -1.90625 0.765625q-0.765625 0.75 -0.84375 2.015625zm9.141342 5.765625l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65
 625 -0.75 2.515625l0 5.28125l-1.640625 0zm13.953842 -1.46875l0.234375 1.453125q-0.6875 0.140625 -1.234375 0.140625q-0.890625 0 -1.390625 -0.28125q-0.484375 -0.28125 -0.6875 -0.734375q-0.203125 -0.46875 -0.203125 -1.9375l0 -5.578125l-1.203125 0l0 -1.265625l1.203125 0l0 -2.390625l1.625 -0.984375l0 3.375l1.65625 0l0 1.265625l-1.65625 0l0 5.671875q0 0.6875 0.078125 0.890625q0.09375 0.203125 0.28125 0.328125q0.203125 0.109375 0.578125 0.109375q0.265625 0 0.71875 -0.0625zm7.323929 1.46875l-1.640625 0l0 -10.453125q-0.59375 0.5625 -1.5625 1.140625q-0.953125 0.5625 -1.71875 0.84375l0 -1.59375q1.375 -0.640625 2.40625 -1.5625q1.03125 -0.921875 1.453125 -1.78125l1.0625 0l0 13.40625z" fill-rule="nonzero"></path><path fill="#ffffff" d="m392.0 74.04724l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m392.0 74.04724l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path fill="#0
 00000" d="m402.23438 100.96724l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65625 -0.75 2.515625l0 5.28125l-1.640625 0zm10.375732 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm4.535431 0l0 -8.40625l-1.453125 0l0 -1.265625l1.453125 0l0 -1.03125q0 -0.96875 0.171875 -1.453125q0.234375 -0.640625 0.828125 -1.03125q0.59375 -0.390625 1.671875 -0.390625q0.6875 0 1.53125 0.15625l-0.25 1.4375q-0.5 -0.09375 -0.953125 -0.09375q-0.75 0 -1.0625 0.328125q-0.3125 0.3125 -0.3125 1.1875l0 0.890625l1.890625 0l0 1.265625l-1.890625 0l0 8.40625l-1.625 0zm4.792694 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -
 9.671875l1.640625 0l0 9.671875l-1.640625 0zm3.691681 -6.59375q0 -2.359375 0.484375 -3.796875q0.484375 -1.453125 1.4375 -2.234375q0.96875 -0.78125 2.421875 -0.78125q1.078125 0 1.890625 0.4375q0.8125 0.421875 1.328125 1.25q0.53125 0.8125 0.828125 1.984375q0.3125 1.15625 0.3125 3.140625q0 2.359375 -0.484375 3.8125q-0.484375 1.4375 -1.453125 2.234375q-0.953125 0.78125 -2.421875 0.78125q-1.921875 0 -3.03125 -1.390625q-1.3125 -1.671875 -1.3125 -5.4375zm1.671875 0q0 3.296875 0.765625 4.390625q0.78125 1.078125 1.90625 1.078125q1.140625 0 1.90625 -1.09375q0.765625 -1.09375 0.765625 -4.375q0 -3.296875 -0.765625 -4.375q-0.765625 -1.078125 -1.921875 -1.078125q-1.125 0 -1.796875 0.953125q-0.859375 1.21875 -0.859375 4.5z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m76.0 104.92913l202.99213 0l0 34.3937l-202.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m76.0 104.92913l202.99213 0l0 34.3937l-202.99213 0z" fill
 -rule="evenodd"></path><path fill="#000000" d="m93.703125 127.985985l-2.203125 0l0 -4.234375q0 -1.34375 -0.140625 -1.734375q-0.140625 -0.40625 -0.453125 -0.625q-0.3125 -0.21875 -0.765625 -0.21875q-0.5625 0 -1.015625 0.3125q-0.453125 0.3125 -0.625 0.828125q-0.171875 0.515625 -0.171875 1.90625l0 3.765625l-2.1875 0l0 -8.296875l2.03125 0l0 1.21875q1.09375 -1.40625 2.734375 -1.40625q0.734375 0 1.328125 0.265625q0.609375 0.25 0.90625 0.65625q0.3125 0.40625 0.4375 0.921875q0.125 0.515625 0.125 1.484375l0 5.15625zm2.2265625 -9.421875l0 -2.03125l2.1875 0l0 2.03125l-2.1875 0zm0 9.421875l0 -8.296875l2.1875 0l0 8.296875l-2.1875 0zm3.4765625 -8.296875l1.21875 0l0 -0.625q0 -1.046875 0.21875 -1.5625q0.234375 -0.515625 0.828125 -0.84375q0.59375 -0.328125 1.515625 -0.328125q0.9375 0 1.828125 0.28125l-0.296875 1.53125q-0.515625 -0.125 -1.0 -0.125q-0.484375 0 -0.6875 0.234375q-0.203125 0.21875 -0.203125 0.84375l0 0.59375l1.640625 0l0 1.71875l-1.640625 0l0 6.578125l-2.203125 0l0 -6.578125l-1.21875 0l0 
 -1.71875zm6.296875 -1.125l0 -2.03125l2.1875 0l0 2.03125l-2.1875 0zm0 9.421875l0 -8.296875l2.1875 0l0 8.296875l-2.1875 0z" fill-rule="nonzero"></path><path fill="#000000" d="m110.44531 127.985985l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.7265625 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.7031174 -2.359375 0.7031174q-1.828125 0 -2.890625 -1.1249924q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.703125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm6.8984375 4.953125l3.03125 -4.3125l-2.8125 -3.984375l1.765625 0l1.265625 1.9375q0.359375 0.5625 0.578125 0.9375q0.34375 -0.515625 0.64
 0625 -0.921875l1.390625 -1.953125l1.6875 0l-2.875 3.90625l3.09375 4.390625l-1.734375 0l-1.703125 -2.578125l-0.453125 -0.703125l-2.171875 3.28125l-1.703125 0zm14.34375 -1.03125q-0.78125 0.671875 -1.5 0.953125q-0.71875 0.26561737 -1.546875 0.26561737q-1.375 0 -2.109375 -0.6718674q-0.734375 -0.671875 -0.734375 -1.703125q0 -0.609375 0.28125 -1.109375q0.28125 -0.515625 0.71875 -0.8125q0.453125 -0.3125 1.015625 -0.46875q0.421875 -0.109375 1.25 -0.203125q1.703125 -0.203125 2.515625 -0.484375q0 -0.296875 0 -0.375q0 -0.859375 -0.390625 -1.203125q-0.546875 -0.484375 -1.609375 -0.484375q-0.984375 0 -1.46875 0.359375q-0.46875 0.34375 -0.6875 1.21875l-1.375 -0.1875q0.1875 -0.875 0.609375 -1.421875q0.4375 -0.546875 1.25 -0.828125q0.8125 -0.296875 1.875 -0.296875q1.0625 0 1.71875 0.25q0.671875 0.25 0.984375 0.625q0.3125 0.375 0.4375 0.953125q0.078125 0.359375 0.078125 1.296875l0 1.875q0 1.96875 0.078125 2.484375q0.09375 0.515625 0.359375 1.0l-1.46875 0q-0.21875 -0.4375 -0.28125 -1.03125zm-0.109375
  -3.140625q-0.765625 0.3125 -2.296875 0.53125q-0.875 0.125 -1.234375 0.28125q-0.359375 0.15625 -0.5625 0.46875q-0.1875 0.296875 -0.1875 0.65625q0 0.5625 0.421875 0.9375q0.4375 0.375 1.25 0.375q0.8125 0 1.4375 -0.34375q0.640625 -0.359375 0.9375 -0.984375q0.234375 -0.46875 0.234375 -1.40625l0 -0.515625zm3.6015625 4.171875l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.84375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm13.328125 3
 .1874924l0 -11.484367l1.28125 0l0 1.078125q0.453125 -0.640625 1.015625 -0.953125q0.578125 -0.3125 1.390625 -0.3125q1.0625 0 1.875 0.546875q0.8125 0.546875 1.21875 1.546875q0.421875 0.984375 0.421875 2.171875q0 1.28125 -0.46875 2.296875q-0.453125 1.015625 -1.328125 1.5625q-0.859375 0.5468674 -1.828125 0.5468674q-0.703125 0 -1.265625 -0.29686737q-0.546875 -0.296875 -0.90625 -0.75l0 4.0468674l-1.40625 0zm1.265625 -7.2968674q0 1.609375 0.640625 2.375q0.65625 0.765625 1.578125 0.765625q0.9375 0 1.609375 -0.796875q0.671875 -0.796875 0.671875 -2.453125q0 -1.59375 -0.65625 -2.375q-0.65625 -0.796875 -1.5625 -0.796875q-0.890625 0 -1.59375 0.84375q-0.6875 0.84375 -0.6875 2.4375zm7.6015625 4.109375l0 -11.453125l1.40625 0l0 11.453125l-1.40625 0zm9.2578125 -2.671875l1.453125 0.171875q-0.34375 1.28125 -1.28125 1.984375q-0.921875 0.7031174 -2.359375 0.7031174q-1.828125 0 -2.890625 -1.1249924q-1.0625 -1.125 -1.0625 -3.140625q0 -2.09375 1.078125 -3.25q1.078125 -1.15625 2.796875 -1.15625q1.65625 0 2.7
 03125 1.140625q1.0625 1.125 1.0625 3.171875q0 0.125 0 0.375l-6.1875 0q0.078125 1.375 0.765625 2.109375q0.703125 0.71875 1.734375 0.71875q0.78125 0 1.328125 -0.40625q0.546875 -0.40625 0.859375 -1.296875zm-4.609375 -2.28125l4.625 0q-0.09375 -1.046875 -0.53125 -1.5625q-0.671875 -0.8125 -1.734375 -0.8125q-0.96875 0 -1.640625 0.65625q-0.65625 0.640625 -0.71875 1.71875zm8.2265625 4.953125l0 -1.609375l1.609375 0l0 1.609375l-1.609375 0zm9.4609375 -3.046875l1.390625 0.1875q-0.234375 1.421875 -1.171875 2.234375q-0.921875 0.8124924 -2.28125 0.8124924q-1.703125 0 -2.75 -1.1093674q-1.03125 -1.125 -1.03125 -3.203125q0 -1.34375 0.4375 -2.34375q0.453125 -1.015625 1.359375 -1.515625q0.921875 -0.5 1.984375 -0.5q1.359375 0 2.21875 0.6875q0.859375 0.671875 1.09375 1.9375l-1.359375 0.203125q-0.203125 -0.828125 -0.703125 -1.25q-0.484375 -0.421875 -1.1875 -0.421875q-1.0625 0 -1.734375 0.765625q-0.65625 0.75 -0.65625 2.40625q0 1.671875 0.640625 2.4375q0.640625 0.75 1.671875 0.75q0.828125 0 1.375 -0.5q0.562
 5 -0.515625 0.703125 -1.578125zm2.0625 -1.109375q0 -2.296875 1.28125 -3.40625q1.078125 -0.921875 2.609375 -0.921875q1.71875 0 2.796875 1.125q1.09375 1.109375 1.09375 3.09375q0 1.59375 -0.484375 2.515625q-0.484375 0.921875 -1.40625 1.4375q-0.90625 0.49999237 -2.0 0.49999237q-1.734375 0 -2.8125 -1.1093674q-1.078125 -1.125 -1.078125 -3.234375zm1.453125 0q0 1.59375 0.6875 2.390625q0.703125 0.796875 1.75 0.796875q1.046875 0 1.734375 -0.796875q0.703125 -0.796875 0.703125 -2.4375q0 -1.53125 -0.703125 -2.328125q-0.6875 -0.796875 -1.734375 -0.796875q-1.046875 0 -1.75 0.796875q-0.6875 0.78125 -0.6875 2.375zm7.9765625 4.15625l0 -8.296875l1.25 0l0 1.15625q0.390625 -0.609375 1.03125 -0.96875q0.65625 -0.375 1.484375 -0.375q0.921875 0 1.515625 0.390625q0.59375 0.375 0.828125 1.0625q0.984375 -1.453125 2.5625 -1.453125q1.234375 0 1.890625 0.6875q0.671875 0.671875 0.671875 2.09375l0 5.703125l-1.390625 0l0 -5.234375q0 -0.84375 -0.140625 -1.203125q-0.140625 -0.375 -0.5 -0.59375q-0.359375 -0.234375 -0.8
 4375 -0.234375q-0.875 0 -1.453125 0.578125q-0.578125 0.578125 -0.578125 1.859375l0 4.828125l-1.40625 0l0 -5.390625q0 -0.9375 -0.34375 -1.40625q-0.34375 -0.46875 -1.125 -0.46875q-0.59375 0 -1.09375 0.3125q-0.5 0.3125 -0.734375 0.921875q-0.21875 0.59375 -0.21875 1.71875l0 4.3125l-1.40625 0zm13.71875 -6.703125l0 -1.59375l1.59375 0l0 1.59375l-1.59375 0zm0 6.703125l0 -1.609375l1.59375 0l0 1.609375l-1.59375 0zm8.1640625 0l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.
 609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.0468674 -2.765625 1.0468674q-1.5 0 -2.5 -0.8906174q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#000000" fill-opacity="0.0" d="m278.99213 122.125984l65.502625 0l0 0.06299591l65.520996 0" fill-rule="evenodd"></path><path stroke="#ff0000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" stroke-dasharray="4.0,3.0" d="m
 278.99213 122.125984l65.502625 0l0 0.06299591l59.520996 0" fill-rule="evenodd"></path><path fill="#ff0000" stroke="#ff0000" stroke-width="1.0" stroke-linecap="butt" d="m404.01575 123.840706l4.538086 -1.6517258l-4.538086 -1.6517334z" fill-rule="evenodd"></path><path fill="#cfe2f3" d="m410.0 109.51181l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m410.0 109.51181l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m421.82812 121.767235q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 0.9999924 -2.734375 0.9999924q-1.6875 0 -2.734375
  -0.9999924q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875
  0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.0468674 -2.765625 1.0468674q-1.5 0 -2.5 -0.8906174q-1.0 -0.90625 -1.140625 -2.343
 75z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m410.0 141.58267l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m410.0 141.58267l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m421.82812 153.8381q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.7
 8125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625
  -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 
 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm12.7734375 3.3125l-1.40625 0l0 -8.96875q-0.515625 0.484375 -1.34375 0.96875q-0.8125 0.484375 -1.46875 0.734375l0 -1.359375q1.171875 -0.5625 2.046875 -1.34375q0.890625 -0.796875 1.265625 -1.53125l0.90625 0l0 11.5z" fill-rule="nonzero"></path><path fill="#ffffff" d="m470.0 135.81102l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m470.0 135.81102l82.99213 0l0 96.944885l-82.99213 0z" fill-rule="evenodd"></path><path fill="#000000" d="m480.23438 162.73102l0 -9.671875l1.46875 0l0 1.375q1.0625 -1.59375 3.078125 -1.59375q0.875 0 1.609375 0.3125q0.734375 0.3125 1.09375 0.828125q0.375 0.5 0.515625 1.203125q0.09375 0.453125 0.09375 1.59375l0 5.953125l-1.640625 0l0 -5.890625q0 -1.0 -0.
 203125 -1.484375q-0.1875 -0.5 -0.671875 -0.796875q-0.484375 -0.296875 -1.140625 -0.296875q-1.046875 0 -1.8125 0.671875q-0.75 0.65625 -0.75 2.515625l0 5.28125l-1.640625 0zm10.375732 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm4.535431 0l0 -8.40625l-1.453125 0l0 -1.265625l1.453125 0l0 -1.03125q0 -0.96875 0.171875 -1.453125q0.234375 -0.640625 0.828125 -1.03125q0.59375 -0.390625 1.671875 -0.390625q0.6875 0 1.53125 0.15625l-0.25 1.4375q-0.5 -0.09375 -0.953125 -0.09375q-0.75 0 -1.0625 0.328125q-0.3125 0.3125 -0.3125 1.1875l0 0.890625l1.890625 0l0 1.265625l-1.890625 0l0 8.40625l-1.625 0zm4.792694 -11.46875l0 -1.890625l1.640625 0l0 1.890625l-1.640625 0zm0 11.46875l0 -9.671875l1.640625 0l0 9.671875l-1.640625 0zm9.863556 0l-1.640625 0l0 -10.453125q-0.59375 0.5625 -1.5625 1.140625q-0.953125 0.5625 -1.71875 0.84375l0 -1.59375q1.375 -0.640625 2.40625 -1.5625q1.03125 -0.921875 1.453125 -1.78125l1.0625 0l0 13.40625z" fill-rule=
 "nonzero"></path><path fill="#cfe2f3" d="m484.10278 172.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m484.10278 172.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m495.9309 184.60188q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0
 .53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm11.9765625 3.3125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm8.8984375 4.03125l0 -2.75l-4.96875 0l0 -1.28125l5.234375 -7.421875l1.140625 0l0 7.421875l1.546875 0l0 1.28125l-1.546875 0l0 2.75l-1.40625 0zm0 -4.03125l0 -5.171875l-3.578125 5.171875l3.578125 0zm4.3984375 1.0l1.40625 -0.1875q0.25 1.203125 0.828125 1.734375q0.578125 0.515625 1.421875 0.515625q0.984375 0 1.671875 -0.6875q0.6875 -0.6875 0.6875 -1.703125q0 -0.96875 -0.640625 
 -1.59375q-0.625 -0.625 -1.609375 -0.625q-0.390625 0 -0.984375 0.15625l0.15625 -1.234375q0.140625 0.015625 0.21875 0.015625q0.90625 0 1.625 -0.46875q0.71875 -0.46875 0.71875 -1.453125q0 -0.765625 -0.53125 -1.265625q-0.515625 -0.515625 -1.34375 -0.515625q-0.828125 0 -1.375 0.515625q-0.546875 0.515625 -0.703125 1.546875l-1.40625 -0.25q0.265625 -1.421875 1.171875 -2.1875q0.921875 -0.78125 2.28125 -0.78125q0.9375 0 1.71875 0.40625q0.796875 0.390625 1.203125 1.09375q0.421875 0.6875 0.421875 1.46875q0 0.75 -0.40625 1.359375q-0.390625 0.609375 -1.171875 0.96875q1.015625 0.234375 1.578125 0.96875q0.5625 0.734375 0.5625 1.84375q0 1.5 -1.09375 2.546875q-1.09375 1.046875 -2.765625 1.046875q-1.5 0 -2.5 -0.890625q-1.0 -0.90625 -1.140625 -2.34375z" fill-rule="nonzero"></path><path fill="#cfe2f3" d="m484.10278 203.34645l57.007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path stroke="#000000" stroke-width="1.0" stroke-linejoin="round" stroke-linecap="butt" d="m484.10278 203.34645l57.
 007874 0l0 25.228348l-57.007874 0z" fill-rule="evenodd"></path><path fill="#000000" d="m495.9309 215.60188q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.515625q0.8125 0 1.328125 -0.515625q0.515625 -0.515625 0.515625 -1.25q0 -0.78125 -0.53125 -1.296875q-0.53125 -0.53125 -1.328125 -0.53125q-0.8125 0 -1.34375 0.515625q-0.53125 0.515625 -0.53125 1.234375zm-0.453125 5.28125q0 0.609375 0.28125 1.171875q0.296875 0.5625 0.859375 0.875q0.5625 0.296875 1.203125 0.296875q1.015625 0 1.
 671875 -0.640625q0.65625 -0.65625 0.65625 -1.65625q0 -1.015625 -0.6875 -1.671875q-0.671875 -0.671875 -1.6875 -0.671875q-0.984375 0 -1.640625 0.65625q-0.65625 0.65625 -0.65625 1.640625zm7.4765625 -2.34375q0 -2.03125 0.40625 -3.265625q0.421875 -1.234375 1.25 -1.90625q0.828125 -0.671875 2.078125 -0.671875q0.921875 0 1.609375 0.375q0.703125 0.359375 1.15625 1.0625q0.453125 0.703125 0.703125 1.703125q0.265625 1.0 0.265625 2.703125q0 2.015625 -0.421875 3.265625q-0.40625 1.234375 -1.234375 1.921875q-0.828125 0.671875 -2.078125 0.671875q-1.65625 0 -2.609375 -1.203125q-1.125 -1.421875 -1.125 -4.65625zm1.4375 0q0 2.828125 0.65625 3.765625q0.671875 0.921875 1.640625 0.921875q0.96875 0 1.625 -0.9375q0.65625 -0.9375 0.65625 -3.75q0 -2.828125 -0.65625 -3.75q-0.65625 -0.9375 -1.640625 -0.9375q-0.96875 0 -1.546875 0.828125q-0.734375 1.046875 -0.734375 3.859375zm9.6171875 -0.5625q-0.875 -0.3125 -1.296875 -0.90625q-0.421875 -0.59375 -0.421875 -1.421875q0 -1.25 0.890625 -2.09375q0.90625 -0.859375 2.40
 625 -0.859375q1.5 0 2.40625 0.875q0.921875 0.859375 0.921875 2.109375q0 0.796875 -0.421875 1.390625q-0.421875 0.59375 -1.265625 0.90625q1.046875 0.34375 1.59375 1.109375q0.5625 0.765625 0.5625 1.828125q0 1.46875 -1.046875 2.484375q-1.03125 1.0 -2.734375 1.0q-1.6875 0 -2.734375 -1.0q-1.03125 -1.015625 -1.03125 -2.515625q0 -1.125 0.5625 -1.875q0.5625 -0.75 1.609375 -1.03125zm-0.28125 -2.375q0 0.8125 0.515625 1.328125q0.53125 0.515625 1.375 0.5156

<TRUNCATED>

[5/5] nifi git commit: NIFI-4932: Enable S2S work behind a Reverse Proxy Adding S2S endpoint Reverse Proxy mapping capability. Added license header to SVG files. Incorporated review comments. Use regex to check property key processing. Catch AttributeExp

Posted by mc...@apache.org.
NIFI-4932: Enable S2S work behind a Reverse Proxy
Adding S2S endpoint Reverse Proxy mapping capability.
Added license header to SVG files.
Incorporated review comments.
Use regex to check property key processing.
Catch AttributeExpressionLanguageParsingException.
This closes #2510


Project: http://git-wip-us.apache.org/repos/asf/nifi/repo
Commit: http://git-wip-us.apache.org/repos/asf/nifi/commit/1913b1e2
Tree: http://git-wip-us.apache.org/repos/asf/nifi/tree/1913b1e2
Diff: http://git-wip-us.apache.org/repos/asf/nifi/diff/1913b1e2

Branch: refs/heads/master
Commit: 1913b1e2a8c798eac066c9ab3baab7843e115ef1
Parents: 7c0ee01
Author: Koji Kawamura <ij...@apache.org>
Authored: Tue Feb 6 11:37:06 2018 +0900
Committer: Matt Gilman <ma...@gmail.com>
Committed: Tue Apr 3 15:40:28 2018 -0400

----------------------------------------------------------------------
 .../src/main/asciidoc/administration-guide.adoc | 257 +++++++++++++++
 .../main/asciidoc/images/s2s-rproxy-http.svg    |  17 +
 .../asciidoc/images/s2s-rproxy-portnumber.svg   |  17 +
 .../asciidoc/images/s2s-rproxy-servername.svg   |  17 +
 .../nifi/remote/PeerDescriptionModifiable.java  |  25 ++
 .../nifi/remote/PeerDescriptionModifier.java    | 182 +++++++++++
 .../nifi/remote/SocketRemoteSiteListener.java   |   5 +
 .../socket/SocketFlowFileServerProtocol.java    |  31 +-
 .../remote/TestPeerDescriptionModifier.java     | 321 +++++++++++++++++++
 .../nifi/web/api/ApplicationResource.java       |  13 +-
 .../nifi/web/api/DataTransferResource.java      |   2 +-
 .../apache/nifi/web/api/SiteToSiteResource.java | 118 +++++--
 .../nifi/web/api/TestDataTransferResource.java  |  51 +++
 13 files changed, 1025 insertions(+), 31 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/nifi/blob/1913b1e2/nifi-docs/src/main/asciidoc/administration-guide.adoc
----------------------------------------------------------------------
diff --git a/nifi-docs/src/main/asciidoc/administration-guide.adoc b/nifi-docs/src/main/asciidoc/administration-guide.adoc
index 0cf4e77..4ad817e 100644
--- a/nifi-docs/src/main/asciidoc/administration-guide.adoc
+++ b/nifi-docs/src/main/asciidoc/administration-guide.adoc
@@ -2659,6 +2659,10 @@ RFC 5952 Sections link:https://tools.ietf.org/html/rfc5952#section-4[4] and link
 _nifi.properties_. This property accepts a comma separated list of expected values. In the event an incoming request has an X-ProxyContextPath or X-Forwarded-Context header value that is not
 present in the whitelist, the "An unexpected error has occurred" page will be shown and an error will be written to the nifi-app.log.
 
+* Additional configurations at both proxy server and NiFi cluster are required to make NiFi Site-to-Site work behind reverse proxies. See <<site_to_site_reverse_proxy_properties>> for details.
+
+** In order to transfer data via Site-to-Site protocol through reverse proxies, both proxy and Site-to-Site client NiFi users need to have following policies, 'retrieve site-to-site details', 'receive data via site-to-site' for input ports, and 'send data via site-to-site' for output ports.
+
 [[kerberos_service]]
 == Kerberos Service
 NiFi can be configured to use Kerberos SPNEGO (or "Kerberos Service") for authentication. In this scenario, users will hit the REST endpoint `/access/kerberos` and the server will respond with a `401` status code and the challenge response header `WWW-Authenticate: Negotiate`. This communicates to the browser to use the GSS-API and load the user's Kerberos ticket and provide it as a Base64-encoded header value in the subsequent request. It will be of the form `Authorization: Negotiate YII...`. NiFi will attempt to validate this ticket with the KDC. If it is successful, the user's _principal_ will be returned as the identity, and the flow will follow login/credential authentication, in that a JWT will be issued in the response to prevent the unnecessary overhead of Kerberos authentication on every subsequent request. If the ticket cannot be validated, it will return with the appropriate error response code. The user will then be able to provide their Kerberos credentials to the login
  form if the `KerberosLoginIdentityProvider` has been configured. See <<kerberos_login_identity_provider>> login identity provider for more details.
@@ -3069,6 +3073,259 @@ responses from the remote system for `30 secs`. This allows NiFi to avoid consta
 has many instances of Remote Process Groups.
 |====
 
+[[site_to_site_reverse_proxy_properties]]
+=== Site to Site Routing Properties for Reverse Proxies
+
+Site-to-Site requires peer-to-peer communication between a client and a remote NiFi node. E.g. if a remote NiFi cluster has 3 nodes, nifi0, nifi1 and nifi2, then a client requests have to be reachable to each of those remote node.
+
+If a NiFi cluster is planned to receive/transfer data from/to Site-to-Site clients over the internet or a company firewall, a reverse proxy server can be deployed in front of the NiFi cluster nodes as a gateway to route client requests to upstream NiFi nodes, to reduce number of servers and ports those have to be exposed.
+
+In such environment, the same NiFi cluster would also be expected to be accessed by Site-to-Site clients within the same network. Sending FlowFiles to itself for load distribution among NiFi cluster nodes can be a typical example. In this case, client requests should be routed directly to a node without going through the reverse proxy.
+
+In order to support such deployments, remote NiFi clusters need to expose its Site-to-Site endpoints dynamically based on client request contexts. Following properties configure how peers should be exposed to clients. A routing definition consists of 4 properties, 'when', 'hostname', 'port', and 'secure', grouped by 'protocol' and 'name'. Multiple routing definitions can be configured. 'protocol' represents Site-to-Site transport protocol, i.e. raw or http.
+
+|====
+|*Property*|*Description*
+|nifi.remote.route.{protocol}.{name}.when|Boolean value, 'true' or 'false'. Controls whether the routing definition for this name should be used.
+|nifi.remote.route.{protocol}.{name}.hostname|Specify hostname that will be introduced to Site-to-Site clients for further communications.
+|nifi.remote.route.{protocol}.{name}.port|Specify port number that will be introduced to Site-to-Site clients for further communications.
+|nifi.remote.route.{protocol}.{name}.secure|Boolean value, 'true' or 'false'. Specify whether the remote peer should be accessed via secure protocol. Defaults to 'false'.
+|====
+
+All of above routing properties can use NiFi Expression Language to compute target peer description from request context. Available variables are:
+
+|===
+|*Variable name*|*Description*
+|s2s.{source\|target}.hostname|Hostname of the source where the request came from, and the original target.
+|s2s.{source\|target}.port|Same as above, for ports. Source port may not be useful as it is just a client side TCP port.
+|s2s.{source\|target}.secure|Same as above, for secure or not.
+|s2s.protocol|The name of Site-to-Site protocol being used, RAW or HTTP.
+|s2s.request|The name of current request type, SiteToSiteDetail or Peers. See Site-to-Site protocol sequence below for detail.
+|HTTP request headers|HTTP request header values can be referred by its name.
+|===
+
+==== Site to Site protocol sequence
+
+Configuring these properties correctly would require some understandings on Site-to-Site protocol sequence.
+
+1. A client initiates Site-to-Site protocol by sending a HTTP(S) request to the specified remote URL to get remote cluster Site-to-Site information. Specifically, to '/nifi-api/site-to-site'. This request is called 'SiteToSiteDetail'.
+2. A remote NiFi node responds with its input and output ports, and TCP port numbers for RAW and TCP transport protocols.
+3. The client sends another request to get remote peers using the TCP port number returned at #2. From this request, raw socket communication is used for RAW transport protocol, while HTTP keeps using HTTP(S). This request is called 'Peers'.
+4. A remote NiFi node responds with list of available remote peers containing hostname, port, secure and workload such as the number of queued FlowFiles. From this point, further communication is done between the client and the remote NiFi node.
+5. The client decides which peer to transfer data from/to, based on workload information.
+6. The client sends a request to create a transaction to a remote NiFi node.
+7. The remote NiFi node accepts the transaction.
+8. Data is sent to the target peer. Multiple Data packets can be sent in batch manner.
+9. When there is no more data to send, or reached to batch limit, the transaction is confirmed on both end by calculating CRC32 hash of sent data.
+10. The transaction is committed on both end.
+
+==== Reverse Proxy Configurations
+
+Most reverse proxy software implement HTTP and TCP proxy mode. For NiFi RAW Site-to-Site protocol, both HTTP and TCP proxy configurations are required, and at least 2 ports needed to be opened. NiFi HTTP Site-to-Site protocol can minimize the required number of open ports at the reverse proxy to 1.
+
+Setting correct HTTP headers at reverse proxies are crucial for NiFi to work correctly, not only routing requests but also authorize client requests. See also <<proxy_configuration>> for details.
+
+There are two types of requests-to-NiFi-node mapping techniques those can be applied at reverse proxy servers. One is 'Server name to Node' and the other is 'Port number to Node'.
+
+With 'Server name to Node', the same port can be used to route requests to different upstream NiFi nodes based on the requested server name (e.g. nifi0.example.com, nifi1.example.com). Host name resolution should be configured to map different host names to the same reverse proxy address, that can be done by adding /etc/hosts file or DNS server entries. Also, if clients to reverse proxy uses HTTPS, reverse proxy server certificate should have wildcard common name or SAN to be accessed by different host names.
+
+Some reverse proxy technologies do not support server name routing rules, in such case, use 'Port number to Node' technique. 'Port number to Node' mapping requires N open port at a reverse proxy for a NiFi cluster consists of N nodes.
+
+Refer following examples for actual configurations.
+
+==== Site to Site and Reverse Proxy Examples
+
+Here are some example reverse proxy and NiFi setups to illustrate how configuration files look like.
+
+Client1 in the following diagrams represents a client that does not have direct access to NiFi nodes, and it accesses through the reverse proxy, while Client2 has direct access.
+
+In this example, Nginx is used as a reverse proxy.
+
+===== Example 1: RAW - Server name to Node mapping
+
+image:s2s-rproxy-servername.svg["Server name to Node mapping"]
+
+1. Client1 initiates Site-to-Site protocol, the request is routed to one of upstream NiFi nodes. The NiFi node computes Site-to-Site port for RAW. By the routing rule 'example1' in nifi.properties shown below, port 10443 is returned.
+2. Client1 asks peers to 'nifi.example.com:10443', the request is routed to 'nifi0:8081'. The NiFi node computes available peers, by 'example1' routing rule, 'nifi0:8081' is converted to 'nifi0.example.com:10443', so are nifi1 and nifi2. As a result, 'nifi0.example.com:10443', 'nifi1.example.com:10443' and 'nifi2.example.com:10443' are returned.
+3. Client1 decides to use 'nifi2.example.com:10443' for further communication.
+4. On the other hand, Client2 has two URIs for Site-to-Site bootstrap URIs, and initiates the protocol using one of them. The 'example1' routing does not match this for this request, and port 8081 is returned.
+5. Client2 asks peers from 'nifi1:8081'. The 'example1' does not match, so the original 'nifi0:8081', 'nifi1:8081' and 'nifi2:8081' are returned as they are.
+6. Client2 decides to use 'nifi2:8081' for further communication.
+
+Routing rule 'example1' is defined in nifi.properties (all node has the same routing configuration):
+....
+# S2S Routing for RAW, using server name to node
+nifi.remote.route.raw.example1.when=\
+${X-ProxyHost:equals('nifi.example.com'):or(\
+${s2s.source.hostname:equals('nifi.example.com'):or(\
+${s2s.source.hostname:equals('192.168.99.100')})})}
+nifi.remote.route.raw.example1.hostname=${s2s.target.hostname}.example.com
+nifi.remote.route.raw.example1.port=10443
+nifi.remote.route.raw.example1.secure=true
+....
+
+
+nginx.conf
+....
+http {
+
+    upstream nifi {
+        server nifi0:8443;
+        server nifi1:8443;
+        server nifi2:8443;
+    }
+
+    # Use dnsmasq so that hostnames such as 'nifi0' can be resolved by /etc/hosts
+    resolver 127.0.0.1;
+
+    server {
+        listen 443 ssl;
+        server_name nifi.example.com;
+        ssl_certificate /etc/nginx/nginx.crt;
+        ssl_certificate_key /etc/nginx/nginx.key;
+
+        proxy_ssl_certificate /etc/nginx/nginx.crt;
+        proxy_ssl_certificate_key /etc/nginx/nginx.key;
+        proxy_ssl_trusted_certificate /etc/nginx/nifi-cert.pem;
+
+        location / {
+            proxy_pass https://nifi;
+            proxy_set_header X-ProxyScheme https;
+            proxy_set_header X-ProxyHost nginx.example.com;
+            proxy_set_header X-ProxyPort 17590;
+            proxy_set_header X-ProxyContextPath /;
+            proxy_set_header X-ProxiedEntitiesChain $ssl_client_s_dn;
+        }
+    }
+}
+
+stream {
+
+    map $ssl_preread_server_name $nifi {
+        nifi0.example.com nifi0;
+        nifi1.example.com nifi1;
+        nifi2.example.com nifi2;
+        default nifi0;
+    }
+
+    resolver 127.0.0.1;
+
+    server {
+        listen 10443;
+        proxy_pass $nifi:8081;
+    }
+}
+....
+
+===== Example 2: RAW - Port number to Node mapping
+
+image:s2s-rproxy-portnumber.svg["Port number to Node mapping"]
+
+The 'example2' routing maps original host names (nifi0, 1 and 2) to different proxy ports (10443, 10444 and 10445) using 'equals and 'ifElse' expressions.
+
+nifi.properties (all node has the same routing configuration)
+....
+# S2S Routing for RAW, using port number to node
+nifi.remote.route.raw.example2.when=\
+${X-ProxyHost:equals('nifi.example.com'):or(\
+${s2s.source.hostname:equals('nifi.example.com'):or(\
+${s2s.source.hostname:equals('192.168.99.100')})})}
+nifi.remote.route.raw.example2.hostname=nifi.example.com
+nifi.remote.route.raw.example2.port=\
+${s2s.target.hostname:equals('nifi0'):ifElse('10443',\
+${s2s.target.hostname:equals('nifi1'):ifElse('10444',\
+${s2s.target.hostname:equals('nifi2'):ifElse('10445',\
+'undefined')})})}
+nifi.remote.route.raw.example2.secure=true
+....
+
+nginx.conf
+....
+http {
+    # Same as example 1.
+}
+
+stream {
+
+    map $ssl_preread_server_name $nifi {
+        nifi0.example.com nifi0;
+        nifi1.example.com nifi1;
+        nifi2.example.com nifi2;
+        default nifi0;
+    }
+
+    resolver 127.0.0.1;
+
+    server {
+        listen 10443;
+        proxy_pass nifi0:8081;
+    }
+    server {
+        listen 10444;
+        proxy_pass nifi1:8081;
+    }
+    server {
+        listen 10445;
+        proxy_pass nifi2:8081;
+    }
+}
+....
+
+===== Example 3: HTTP - Server name to Node mapping
+
+image:s2s-rproxy-http.svg["Server name to Node mapping"]
+
+nifi.properties (all node has the same routing configuration)
+....
+# S2S Routing for HTTP
+nifi.remote.route.http.example3.when=${X-ProxyHost:contains('.example.com')}
+nifi.remote.route.http.example3.hostname=${s2s.target.hostname}.example.com
+nifi.remote.route.http.example3.port=443
+nifi.remote.route.http.example3.secure=true
+....
+
+nginx.conf
+....
+http {
+    upstream nifi_cluster {
+        server nifi0:8443;
+        server nifi1:8443;
+        server nifi2:8443;
+    }
+
+    # If target node is not specified, use one from cluster.
+    map $http_host $nifi {
+        nifi0.example.com:443 "nifi0:8443";
+        nifi1.example.com:443 "nifi1:8443";
+        nifi2.example.com:443 "nifi2:8443";
+        default "nifi_cluster";
+    }
+
+    resolver 127.0.0.1;
+
+    server {
+        listen 443 ssl;
+        server_name ~^(.+\.example\.com)$;
+        ssl_certificate /etc/nginx/nginx.crt;
+        ssl_certificate_key /etc/nginx/nginx.key;
+
+        proxy_ssl_certificate /etc/nginx/nginx.crt;
+        proxy_ssl_certificate_key /etc/nginx/nginx.key;
+        proxy_ssl_trusted_certificate /etc/nginx/nifi-cert.pem;
+
+        location / {
+            proxy_pass https://$nifi;
+            proxy_set_header X-ProxyScheme https;
+            proxy_set_header X-ProxyHost $1;
+            proxy_set_header X-ProxyPort 443;
+            proxy_set_header X-ProxyContextPath /;
+            proxy_set_header X-ProxiedEntitiesChain $ssl_client_s_dn;
+        }
+    }
+}
+....
+
+
 === Web Properties
 
 These properties pertain to the web-based User Interface.