From 82137874ea48a5dd2f6e2b6283da634ed8417c43 Mon Sep 17 00:00:00 2001 From: Jonathan Wei Date: Wed, 30 Jan 2019 19:41:07 -0800 Subject: [PATCH] Add master/data/query server concepts to docs/packaging (#6916) * Add master/data/query server concepts to docs/packaging * PR comments * TOC and markdown fix * Update image legend * PR comment * More PR comments --- docs/content/comparisons/druid-vs-redshift.md | 2 +- docs/content/configuration/index.md | 586 +++++++++--------- docs/content/configuration/realtime.md | 2 +- .../dependencies/cassandra-deep-storage.md | 4 +- docs/content/dependencies/metadata-storage.md | 3 +- docs/content/dependencies/zookeeper.md | 6 +- docs/content/design/auth.md | 2 +- docs/content/design/broker.md | 6 +- docs/content/design/coordinator.md | 40 +- docs/content/design/historical.md | 14 +- docs/content/design/index.md | 97 ++- docs/content/design/indexing-service.md | 4 +- docs/content/design/middlemanager.md | 4 +- docs/content/design/overlord.md | 14 +- docs/content/design/peons.md | 2 +- docs/content/design/processes.md | 129 ++++ .../extensions-contrib/materialized-view.md | 4 +- .../extensions-core/druid-basic-security.md | 18 +- .../extensions-core/druid-kerberos.md | 4 +- .../extensions-core/kafka-ingestion.md | 16 +- .../extensions-core/kinesis-ingestion.md | 16 +- .../extensions-core/lookups-cached-global.md | 8 +- .../content/development/extensions-core/s3.md | 2 +- docs/content/development/javascript.md | 2 +- docs/content/development/modules.md | 8 +- docs/content/development/overview.md | 2 +- docs/content/development/router.md | 56 +- docs/content/ingestion/faq.md | 10 +- docs/content/ingestion/hadoop.md | 4 +- docs/content/ingestion/index.md | 4 +- .../content/ingestion/locking-and-priority.md | 4 +- docs/content/ingestion/native_tasks.md | 6 +- docs/content/ingestion/reports.md | 4 +- docs/content/ingestion/stream-pull.md | 2 +- docs/content/ingestion/stream-push.md | 6 +- docs/content/ingestion/tasks.md | 2 +- .../content/ingestion/update-existing-data.md | 4 +- docs/content/operations/api-reference.md | 171 ++--- docs/content/operations/metrics.md | 10 +- docs/content/operations/other-hadoop.md | 4 +- docs/content/operations/performance-faq.md | 6 +- docs/content/operations/recommendations.md | 2 +- docs/content/operations/rolling-updates.md | 4 +- docs/content/operations/rule-configuration.md | 14 +- .../operations/segment-optimization.md | 6 +- docs/content/querying/caching.md | 2 +- docs/content/querying/datasource.md | 2 +- docs/content/querying/dimensionspecs.md | 4 +- docs/content/querying/groupbyquery.md | 36 +- docs/content/querying/lookups.md | 32 +- docs/content/querying/multitenancy.md | 6 +- docs/content/querying/query-context.md | 10 +- docs/content/querying/querying.md | 4 +- docs/content/querying/scan-query.md | 2 +- docs/content/querying/searchquery.md | 4 +- docs/content/querying/segmentmetadataquery.md | 4 +- docs/content/querying/sql.md | 50 +- docs/content/querying/topnquery.md | 2 +- docs/content/toc.md | 11 +- docs/content/tutorials/cluster.md | 79 ++- docs/content/tutorials/tutorial-batch.md | 10 +- docs/content/tutorials/tutorial-compaction.md | 6 +- .../content/tutorials/tutorial-delete-data.md | 2 +- docs/content/tutorials/tutorial-query.md | 4 +- docs/img/druid-architecture.png | Bin 151681 -> 207086 bytes 65 files changed, 876 insertions(+), 708 deletions(-) create mode 100644 docs/content/design/processes.md diff --git a/docs/content/comparisons/druid-vs-redshift.md b/docs/content/comparisons/druid-vs-redshift.md index de74a69472b..86e20768444 100644 --- a/docs/content/comparisons/druid-vs-redshift.md +++ b/docs/content/comparisons/druid-vs-redshift.md @@ -42,7 +42,7 @@ Druid’s write semantics are not as fluid and does not support full joins (we s ### Data distribution model -Druid’s data distribution is segment-based and leverages a highly available "deep" storage such as S3 or HDFS. Scaling up (or down) does not require massive copy actions or downtime; in fact, losing any number of historical nodes does not result in data loss because new historical nodes can always be brought up by reading data from "deep" storage. +Druid’s data distribution is segment-based and leverages a highly available "deep" storage such as S3 or HDFS. Scaling up (or down) does not require massive copy actions or downtime; in fact, losing any number of Historical nodes does not result in data loss because new Historical nodes can always be brought up by reading data from "deep" storage. To contrast, ParAccel’s data distribution model is hash-based. Expanding the cluster requires re-hashing the data across the nodes, making it difficult to perform without taking downtime. Amazon’s Redshift works around this issue with a multi-step process: diff --git a/docs/content/configuration/index.md b/docs/content/configuration/index.md index 0b114a4e6d1..990f9ce79c9 100644 --- a/docs/content/configuration/index.md +++ b/docs/content/configuration/index.md @@ -43,44 +43,48 @@ This page documents all of the configuration properties for each Druid service t * [Metadata Storage](#metadata-storage) * [Deep Storage](#deep-storage) * [Task Logging](#task-logging) - * [Indexing Service Discovery](#indexing-service-discovery) + * [Overlord Discovery](#overlord-discovery) * [Coordinator Discovery](#coordinator-discovery) * [Announcing Segments](#announcing-segments) * [JavaScript](#javascript) * [Double Column Storage](#double-column-storage) - * [Coordinator](#coordinator) - * [Static Configuration](#static-configuration) - * [Node Config](#coordinator-node-config) - * [Coordinator Operation](#coordinator-operation) - * [Segment Management](#segment-management) - * [Metadata Retrieval](#metadata-retrieval) - * [Dynamic Configuration](#dynamic-configuration) - * [Lookups](#lookups-dynamic-configuration) - * [Compaction](#compaction-dynamic-configuration) - * [Overlord](#overlord) - * [Node Config](#overlord-node-config) - * [Static Configuration](#overlord-static-configuration) - * [Dynamic Configuration](#overlord-dynamic-configuration) - * [Worker Select Strategy](#worker-select-strategy) - * [Autoscaler](#autoscaler) - * [MiddleManager & Peons](#middlemanager-and-peons) - * [Node Config](#middlemanager-node-config) - * [MiddleManager Configuration](#middlemanager-configuration) - * [Peon Processing](#peon-processing) - * [Peon Query Configuration](#peon-query-configuration) - * [Caching](#peon-caching) - * [Additional Peon Configuration](#additional-peon-configuration) - * [Broker](#broker) - * [Node Config](#broker-node-configs) - * [Query Configuration](#broker-query-configuration) - * [SQL](#sql) - * [Caching](#broker-caching) - * [Segment Discovery](#segment-discovery) - * [Historical](#historical) - * [Node Configuration](#historical-node-config) - * [General Configuration](#historical-general-configuration) - * [Query Configs](#historical-query-configs) - * [Caching](#historical-caching) + * [Master Server](#master-server) + * [Coordinator](#coordinator) + * [Static Configuration](#static-configuration) + * [Node Config](#coordinator-node-config) + * [Coordinator Operation](#coordinator-operation) + * [Segment Management](#segment-management) + * [Metadata Retrieval](#metadata-retrieval) + * [Dynamic Configuration](#dynamic-configuration) + * [Lookups](#lookups-dynamic-configuration) + * [Compaction](#compaction-dynamic-configuration) + * [Overlord](#overlord) + * [Static Configuration](#overlord-static-configuration) + * [Node Config](#overlord-node-config) + * [Overlord Operations](#overlord-operations) + * [Dynamic Configuration](#overlord-dynamic-configuration) + * [Worker Select Strategy](#worker-select-strategy) + * [Autoscaler](#autoscaler) + * [Data Server](#data-server) + * [MiddleManager & Peons](#middlemanager-and-peons) + * [Node Config](#middlemanager-node-config) + * [MiddleManager Configuration](#middlemanager-configuration) + * [Peon Processing](#peon-processing) + * [Peon Query Configuration](#peon-query-configuration) + * [Caching](#peon-caching) + * [Additional Peon Configuration](#additional-peon-configuration) + * [Historical](#historical) + * [Node Configuration](#historical-node-config) + * [General Configuration](#historical-general-configuration) + * [Query Configs](#historical-query-configs) + * [Caching](#historical-caching) + * [Query Server](#query-server) + * [Broker](#broker) + * [Node Config](#broker-node-configs) + * [Query Configuration](#broker-query-configuration) + * [SQL](#sql) + * [Caching](#broker-caching) + * [Segment Discovery](#segment-discovery) * [Caching](#cache-configuration) * [General Query Configuration](#general-query-configuration) * [Realtime nodes (Deprecated)](#realtime-nodes) @@ -118,7 +122,7 @@ conf/tranquility: kafka.json server.json ``` -Each directory has a `runtime.properties` file containing configuration properties for the specific Druid service correponding to the directory (e.g., `historical`). +Each directory has a `runtime.properties` file containing configuration properties for the specific Druid process correponding to the directory (e.g., `historical`). The `jvm.config` files contain JVM flags such as heap sizing properties for each service. @@ -185,8 +189,8 @@ Druid interacts with ZK through a set of standard path configurations. We recomm |`druid.zk.paths.propertiesPath`|Zookeeper properties path.|`${druid.zk.paths.base}/properties`| |`druid.zk.paths.announcementsPath`|Druid node announcement path.|`${druid.zk.paths.base}/announcements`| |`druid.zk.paths.liveSegmentsPath`|Current path for where Druid nodes announce their segments.|`${druid.zk.paths.base}/segments`| -|`druid.zk.paths.loadQueuePath`|Entries here cause historical nodes to load and drop segments.|`${druid.zk.paths.base}/loadQueue`| -|`druid.zk.paths.coordinatorPath`|Used by the coordinator for leader election.|`${druid.zk.paths.base}/coordinator`| +|`druid.zk.paths.loadQueuePath`|Entries here cause Historical nodes to load and drop segments.|`${druid.zk.paths.base}/loadQueue`| +|`druid.zk.paths.coordinatorPath`|Used by the Coordinator for leader election.|`${druid.zk.paths.base}/coordinator`| |`druid.zk.paths.servedSegmentsPath`|@Deprecated. Legacy path for where Druid nodes announce their segments.|`${druid.zk.paths.base}/servedSegments`| The indexing service also uses its own set of paths. These configs can be included in the common configuration. @@ -195,7 +199,7 @@ The indexing service also uses its own set of paths. These configs can be includ |--------|-----------|-------| |`druid.zk.paths.indexer.base`|Base zookeeper path for |`${druid.zk.paths.base}/indexer`| |`druid.zk.paths.indexer.announcementsPath`|Middle managers announce themselves here.|`${druid.zk.paths.indexer.base}/announcements`| -|`druid.zk.paths.indexer.tasksPath`|Used to assign tasks to middle managers.|`${druid.zk.paths.indexer.base}/tasks`| +|`druid.zk.paths.indexer.tasksPath`|Used to assign tasks to MiddleManagers.|`${druid.zk.paths.indexer.base}/tasks`| |`druid.zk.paths.indexer.statusPath`|Parent path for announcement of task statuses.|`${druid.zk.paths.indexer.base}/status`| If `druid.zk.paths.base` and `druid.zk.paths.indexer.base` are both set, and none of the other `druid.zk.paths.*` or `druid.zk.paths.indexer.*` values are set, then the other properties will be evaluated relative to their respective `base`. @@ -488,7 +492,7 @@ To use graphite as emitter set `druid.emitter=graphite`. For configuration detai ### Metadata Storage -These properties specify the jdbc connection and other configuration around the metadata storage. The only processes that connect to the metadata storage with these properties are the [Coordinator](../design/coordinator.html), [Indexing service](../design/indexing-service.html) and [Realtime Nodes](../design/realtime.html). +These properties specify the jdbc connection and other configuration around the metadata storage. The only processes that connect to the metadata storage with these properties are the [Coordinator](../design/coordinator.html), [Overlord](../design/overlord.html) and [Realtime Nodes](../design/realtime.html). |Property|Description|Default| |--------|-----------|-------| @@ -540,7 +544,7 @@ This deep storage is used to interface with Amazon's S3. Note that the `druid-s3 |`druid.storage.bucket`|S3 bucket name.|none| |`druid.storage.baseKey`|S3 object key prefix for storage.|none| |`druid.storage.disableAcl`|Boolean flag for ACL.|false| -|`druid.storage.archiveBucket`|S3 bucket name for archiving when running the indexing-service *archive task*.|none| +|`druid.storage.archiveBucket`|S3 bucket name for archiving when running the *archive task*.|none| |`druid.storage.archiveBaseKey`|S3 object key prefix for archiving.|none| |`druid.storage.useS3aSchema`|If true, use the "s3a" filesystem when using Hadoop-based ingestion. If false, the "s3n" filesystem will be used. Only affects Hadoop-based ingestion.|false| @@ -575,9 +579,9 @@ Caution: Automatic log file deletion typically works based on log file modificat |Property|Description|Default| |--------|-----------|-------| -|`druid.indexer.logs.kill.enabled`|Boolean value for whether to enable deletion of old task logs. If set to true, overlord will submit kill tasks periodically based on `druid.indexer.logs.kill.delay` specified, which will delete task logs from the log directory as well as tasks and tasklogs table entries in metadata storage except for tasks created in the last `druid.indexer.logs.kill.durationToRetain` period. |false| +|`druid.indexer.logs.kill.enabled`|Boolean value for whether to enable deletion of old task logs. If set to true, Overlord will submit kill tasks periodically based on `druid.indexer.logs.kill.delay` specified, which will delete task logs from the log directory as well as tasks and tasklogs table entries in metadata storage except for tasks created in the last `druid.indexer.logs.kill.durationToRetain` period. |false| |`druid.indexer.logs.kill.durationToRetain`| Required if kill is enabled. In milliseconds, task logs and entries in task-related metadata storage tables to be retained created in last x milliseconds. |None| -|`druid.indexer.logs.kill.initialDelay`| Optional. Number of milliseconds after overlord start when first auto kill is run. |random value less than 300000 (5 mins)| +|`druid.indexer.logs.kill.initialDelay`| Optional. Number of milliseconds after Overlord start when first auto kill is run. |random value less than 300000 (5 mins)| |`druid.indexer.logs.kill.delay`|Optional. Number of milliseconds of delay between successive executions of auto kill run. |21600000 (6 hours)| #### File Task Logs @@ -625,13 +629,13 @@ Store task logs in HDFS. Note that the `druid-hdfs-storage` extension must be lo |--------|-----------|-------| |`druid.indexer.logs.directory`|The directory to store logs.|none| -### Indexing Service Discovery +### Overlord Discovery -This config is used to find the [Indexing Service](../design/indexing-service.html) using Curator service discovery. Only required if you are actually running an indexing service. +This config is used to find the [Overlord](../design/overlord.html) using Curator service discovery. Only required if you are actually running an Overlord. |Property|Description|Default| |--------|-----------|-------| -|`druid.selectors.indexing.serviceName`|The druid.service name of the indexing service Overlord node. To start the Overlord with a different name, set it with this property. |druid/overlord| +|`druid.selectors.indexing.serviceName`|The druid.service name of the Overlord node. To start the Overlord with a different name, set it with this property. |druid/overlord| ### Coordinator Discovery @@ -640,7 +644,7 @@ This config is used to find the [Coordinator](../design/coordinator.html) using |Property|Description|Default| |--------|-----------|-------| -|`druid.selectors.coordinator.serviceName`|The druid.service name of the coordinator node. To start the Coordinator with a different name, set it with this property. |druid/coordinator| +|`druid.selectors.coordinator.serviceName`|The druid.service name of the Coordinator node. To start the Coordinator with a different name, set it with this property. |druid/coordinator| ### Announcing Segments @@ -655,8 +659,8 @@ In current Druid, multiple data segments may be announced under the same Znode. |--------|-----------|-------| |`druid.announcer.segmentsPerNode`|Each Znode contains info for up to this many segments.|50| |`druid.announcer.maxBytesPerNode`|Max byte size for Znode.|524288| -|`druid.announcer.skipDimensionsAndMetrics`|Skip Dimensions and Metrics list from segment announcements. NOTE: Enabling this will also remove the dimensions and metrics list from coordinator and broker endpoints.|false| -|`druid.announcer.skipLoadSpec`|Skip segment LoadSpec from segment announcements. NOTE: Enabling this will also remove the loadspec from coordinator and broker endpoints.|false| +|`druid.announcer.skipDimensionsAndMetrics`|Skip Dimensions and Metrics list from segment announcements. NOTE: Enabling this will also remove the dimensions and metrics list from Coordinator and Broker endpoints.|false| +|`druid.announcer.skipLoadSpec`|Skip segment LoadSpec from segment announcements. NOTE: Enabling this will also remove the loadspec from Coordinator and Broker endpoints.|false| ### JavaScript @@ -685,15 +689,19 @@ Support for 64-bit floating point columns was released in Druid 0.11.0, so if yo |--------|-----------|-------| |`druid.indexing.doubleStorage`|Set to "float" to use 32-bit double representation for double columns.|double| -## Coordinator +## Master Server + +This section contains the configuration options for the processes that reside on Master servers (Coordinators and Overlords) in the suggested [three-server configuration](../design/processes.html#server-types). + +### Coordinator For general Coordinator Node information, see [here](../design/coordinator.html). -### Static Configuration +#### Static Configuration -These coordinator static configurations can be defined in the `coordinator/runtime.properties` file. +These Coordinator static configurations can be defined in the `coordinator/runtime.properties` file. -#### Coordinator Node Config +##### Coordinator Node Config |Property|Description|Default| |--------|-----------|-------| @@ -703,51 +711,51 @@ These coordinator static configurations can be defined in the `coordinator/runti |`druid.tlsPort`|TLS port for HTTPS connector, if [druid.enableTlsPort](../operations/tls-support.html) is set then this config will be used. If `druid.host` contains port then that port will be ignored. This should be a non-negative Integer.|8281| |`druid.service`|The name of the service. This is used as a dimension when emitting metrics and alerts to differentiate between the various services|druid/coordinator| -#### Coordinator Operation +##### Coordinator Operation |Property|Description|Default| |--------|-----------|-------| -|`druid.coordinator.period`|The run period for the coordinator. The coordinator’s operates by maintaining the current state of the world in memory and periodically looking at the set of segments available and segments being served to make decisions about whether any changes need to be made to the data topology. This property sets the delay between each of these runs.|PT60S| +|`druid.coordinator.period`|The run period for the Coordinator. The Coordinator’s operates by maintaining the current state of the world in memory and periodically looking at the set of segments available and segments being served to make decisions about whether any changes need to be made to the data topology. This property sets the delay between each of these runs.|PT60S| |`druid.coordinator.period.indexingPeriod`|How often to send compact/merge/conversion tasks to the indexing service. It's recommended to be longer than `druid.manager.segments.pollDuration`|PT1800S (30 mins)| |`druid.coordinator.startDelay`|The operation of the Coordinator works on the assumption that it has an up-to-date view of the state of the world when it runs, the current ZK interaction code, however, is written in a way that doesn’t allow the Coordinator to know for a fact that it’s done loading the current state of the world. This delay is a hack to give it enough time to believe that it has all the data.|PT300S| -|`druid.coordinator.merge.on`|Boolean flag for whether or not the coordinator should try and merge small segments into a more optimal segment size.|false| -|`druid.coordinator.load.timeout`|The timeout duration for when the coordinator assigns a segment to a historical node.|PT15M| -|`druid.coordinator.kill.pendingSegments.on`|Boolean flag for whether or not the coordinator clean up old entries in the `pendingSegments` table of metadata store. If set to true, coordinator will check the created time of most recently complete task. If it doesn't exist, it finds the created time of the earlist running/pending/waiting tasks. Once the created time is found, then for all dataSources not in the `killPendingSegmentsSkipList` (see [Dynamic configuration](#dynamic-configuration)), coordinator will ask the overlord to clean up the entries 1 day or more older than the found created time in the `pendingSegments` table. This will be done periodically based on `druid.coordinator.period` specified.|false| -|`druid.coordinator.kill.on`|Boolean flag for whether or not the coordinator should submit kill task for unused segments, that is, hard delete them from metadata store and deep storage. If set to true, then for all whitelisted dataSources (or optionally all), coordinator will submit tasks periodically based on `period` specified. These kill tasks will delete all segments except for the last `durationToRetain` period. Whitelist or All can be set via dynamic configuration `killAllDataSources` and `killDataSourceWhitelist` described later.|false| +|`druid.coordinator.merge.on`|Boolean flag for whether or not the Coordinator should try and merge small segments into a more optimal segment size.|false| +|`druid.coordinator.load.timeout`|The timeout duration for when the Coordinator assigns a segment to a Historical node.|PT15M| +|`druid.coordinator.kill.pendingSegments.on`|Boolean flag for whether or not the Coordinator clean up old entries in the `pendingSegments` table of metadata store. If set to true, Coordinator will check the created time of most recently complete task. If it doesn't exist, it finds the created time of the earlist running/pending/waiting tasks. Once the created time is found, then for all dataSources not in the `killPendingSegmentsSkipList` (see [Dynamic configuration](#dynamic-configuration)), Coordinator will ask the Overlord to clean up the entries 1 day or more older than the found created time in the `pendingSegments` table. This will be done periodically based on `druid.coordinator.period` specified.|false| +|`druid.coordinator.kill.on`|Boolean flag for whether or not the Coordinator should submit kill task for unused segments, that is, hard delete them from metadata store and deep storage. If set to true, then for all whitelisted dataSources (or optionally all), Coordinator will submit tasks periodically based on `period` specified. These kill tasks will delete all segments except for the last `durationToRetain` period. Whitelist or All can be set via dynamic configuration `killAllDataSources` and `killDataSourceWhitelist` described later.|false| |`druid.coordinator.kill.period`|How often to send kill tasks to the indexing service. Value must be greater than `druid.coordinator.period.indexingPeriod`. Only applies if kill is turned on.|P1D (1 Day)| |`druid.coordinator.kill.durationToRetain`| Do not kill segments in last `durationToRetain`, must be greater or equal to 0. Only applies and MUST be specified if kill is turned on. Note that default value is invalid.|PT-1S (-1 seconds)| |`druid.coordinator.kill.maxSegments`|Kill at most n segments per kill task submission, must be greater than 0. Only applies and MUST be specified if kill is turned on. Note that default value is invalid.|0| -|`druid.coordinator.balancer.strategy`|Specify the type of balancing strategy that the coordinator should use to distribute segments among the historicals. `cachingCost` is logically equivalent to `cost` but is more CPU-efficient on large clusters and will replace `cost` in the future versions, users are invited to try it. Use `diskNormalized` to distribute segments among nodes so that the disks fill up uniformly and use `random` to randomly pick nodes to distribute segments.|`cost`| +|`druid.coordinator.balancer.strategy`|Specify the type of balancing strategy that the Coordinator should use to distribute segments among the Historicals. `cachingCost` is logically equivalent to `cost` but is more CPU-efficient on large clusters and will replace `cost` in the future versions, users are invited to try it. Use `diskNormalized` to distribute segments among nodes so that the disks fill up uniformly and use `random` to randomly pick nodes to distribute segments.|`cost`| |`druid.coordinator.loadqueuepeon.repeatDelay`|The start and repeat delay for the loadqueuepeon , which manages the load and drop of segments.|PT0.050S (50 ms)| -|`druid.coordinator.asOverlord.enabled`|Boolean value for whether this coordinator node should act like an overlord as well. This configuration allows users to simplify a druid cluster by not having to deploy any standalone overlord nodes. If set to true, then overlord console is available at `http://coordinator-host:port/console.html` and be sure to set `druid.coordinator.asOverlord.overlordService` also. See next.|false| +|`druid.coordinator.asOverlord.enabled`|Boolean value for whether this Coordinator node should act like an Overlord as well. This configuration allows users to simplify a druid cluster by not having to deploy any standalone Overlord nodes. If set to true, then Overlord console is available at `http://coordinator-host:port/console.html` and be sure to set `druid.coordinator.asOverlord.overlordService` also. See next.|false| |`druid.coordinator.asOverlord.overlordService`| Required, if `druid.coordinator.asOverlord.enabled` is `true`. This must be same value as `druid.service` on standalone Overlord nodes and `druid.selectors.indexing.serviceName` on Middle Managers.|NULL| -#### Segment Management +##### Segment Management |Property|Possible Values|Description|Default| |--------|---------------|-----------|-------| |`druid.serverview.type`|batch or http|Segment discovery method to use. "http" enables discovering segments using HTTP instead of zookeeper.|batch| -|`druid.coordinator.loadqueuepeon.type`|curator or http|Whether to use "http" or "curator" implementation to assign segment loads/drops to historical|curator| +|`druid.coordinator.loadqueuepeon.type`|curator or http|Whether to use "http" or "curator" implementation to assign segment loads/drops to Historical|curator| -##### Additional config when "http" loadqueuepeon is used +###### Additional config when "http" loadqueuepeon is used |Property|Description|Default| |--------|-----------|-------| -|`druid.coordinator.loadqueuepeon.http.batchSize`|Number of segment load/drop requests to batch in one HTTP request. Note that it must be smaller than `druid.segmentCache.numLoadingThreads` config on historical node.|1| +|`druid.coordinator.loadqueuepeon.http.batchSize`|Number of segment load/drop requests to batch in one HTTP request. Note that it must be smaller than `druid.segmentCache.numLoadingThreads` config on Historical node.|1| -#### Metadata Retrieval +##### Metadata Retrieval |Property|Description|Default| |--------|-----------|-------| |`druid.manager.config.pollDuration`|How often the manager polls the config table for updates.|PT1M| -|`druid.manager.segments.pollDuration`|The duration between polls the Coordinator does for updates to the set of active segments. Generally defines the amount of lag time it can take for the coordinator to notice new segments.|PT1M| -|`druid.manager.rules.pollDuration`|The duration between polls the Coordinator does for updates to the set of active rules. Generally defines the amount of lag time it can take for the coordinator to notice rules.|PT1M| +|`druid.manager.segments.pollDuration`|The duration between polls the Coordinator does for updates to the set of active segments. Generally defines the amount of lag time it can take for the Coordinator to notice new segments.|PT1M| +|`druid.manager.rules.pollDuration`|The duration between polls the Coordinator does for updates to the set of active rules. Generally defines the amount of lag time it can take for the Coordinator to notice rules.|PT1M| |`druid.manager.rules.defaultTier`|The default tier from which default rules will be loaded from.|_default| |`druid.manager.rules.alertThreshold`|The duration after a failed poll upon which an alert should be emitted.|PT10M| -### Dynamic Configuration +#### Dynamic Configuration -The coordinator has dynamic configuration to change certain behaviour on the fly. The coordinator uses a JSON spec object from the Druid [metadata storage](../dependencies/metadata-storage.html) config table. This object is detailed below: +The Coordinator has dynamic configuration to change certain behaviour on the fly. The Coordinator uses a JSON spec object from the Druid [metadata storage](../dependencies/metadata-storage.html) config table. This object is detailed below: -It is recommended that you use the Coordinator Console to configure these parameters. However, if you need to do it via HTTP, the JSON object can be submitted to the coordinator via a POST request at: +It is recommended that you use the Coordinator Console to configure these parameters. However, if you need to do it via HTTP, the JSON object can be submitted to the Coordinator via a POST request at: ``` http://:/druid/coordinator/v1/config @@ -760,7 +768,7 @@ Optional Header Parameters for auditing the config change can also be specified. |`X-Druid-Author`| author making the config change|""| |`X-Druid-Comment`| comment describing the change being done|""| -A sample coordinator dynamic config JSON object is shown below: +A sample Coordinator dynamic config JSON object is shown below: ```json { @@ -779,11 +787,11 @@ Issuing a GET request at the same URL will return the spec that is currently in |Property|Description|Default| |--------|-----------|-------| -|`millisToWaitBeforeDeleting`|How long does the coordinator need to be active before it can start removing (marking unused) segments in metadata storage.|900000 (15 mins)| +|`millisToWaitBeforeDeleting`|How long does the Coordinator need to be active before it can start removing (marking unused) segments in metadata storage.|900000 (15 mins)| |`mergeBytesLimit`|The maximum total uncompressed size in bytes of segments to merge.|524288000L| |`mergeSegmentsLimit`|The maximum number of segments that can be in a single [append task](../ingestion/tasks.html).|100| |`maxSegmentsToMove`|The maximum number of segments that can be moved at any given time.|5| -|`replicantLifetime`|The maximum number of coordinator runs for a segment to be replicated before we start alerting.|15| +|`replicantLifetime`|The maximum number of Coordinator runs for a segment to be replicated before we start alerting.|15| |`replicationThrottleLimit`|The maximum number of segments that can be replicated at one time.|10| |`balancerComputeThreads`|Thread pool size for computing moving cost of segments in segment balancing. Consider increasing this if you have a lot of segments and moving segment starts to get stuck.|1| |`emitBalancingStats`|Boolean flag for whether or not we should emit balancing stats. This is an expensive operation.|false| @@ -792,21 +800,21 @@ Issuing a GET request at the same URL will return the spec that is currently in |`killPendingSegmentsSkipList`|List of dataSources for which pendingSegments are _NOT_ cleaned up if property `druid.coordinator.kill.pendingSegments.on` is true. This can be a list of comma-separated dataSources or a JSON array.|none| |`maxSegmentsInNodeLoadingQueue`|The maximum number of segments that could be queued for loading to any given server. This parameter could be used to speed up segments loading process, especially if there are "slow" nodes in the cluster (with low loading speed) or if too much segments scheduled to be replicated to some particular node (faster loading could be preferred to better segments distribution). Desired value depends on segments loading speed, acceptable replication time and number of nodes. Value 1000 could be a start point for a rather big cluster. Default value is 0 (loading queue is unbounded) |0| -To view the audit history of coordinator dynamic config issue a GET request to the URL - +To view the audit history of Coordinator dynamic config issue a GET request to the URL - ``` http://:/druid/coordinator/v1/config/history?interval= ``` -default value of interval can be specified by setting `druid.audit.manager.auditHistoryMillis` (1 week if not configured) in coordinator runtime.properties +default value of interval can be specified by setting `druid.audit.manager.auditHistoryMillis` (1 week if not configured) in Coordinator runtime.properties -To view last entries of the audit history of coordinator dynamic config issue a GET request to the URL - +To view last entries of the audit history of Coordinator dynamic config issue a GET request to the URL - ``` http://:/druid/coordinator/v1/config/history?count= ``` -#### Lookups Dynamic Configuration (EXPERIMENTAL) +##### Lookups Dynamic Configuration (EXPERIMENTAL) These configuration options control the behavior of the Lookup dynamic configuration described in the [lookups page](../querying/lookups.html) |Property|Description|Default| @@ -818,9 +826,9 @@ These configuration options control the behavior of the Lookup dynamic configura |`druid.manager.lookups.threadPoolSize`|How many nodes can be managed concurrently (concurrent POST and DELETE requests). Requests this limit will wait in a queue until a slot becomes available.|10| |`druid.manager.lookups.period`|How many milliseconds between checks for configuration changes|30_000| -#### Compaction Dynamic Configuration +##### Compaction Dynamic Configuration -Compaction configurations can also be set or updated dynamically without restarting coordinators. For segment compaction, +Compaction configurations can also be set or updated dynamically without restarting Coordinators. For segment compaction, please see [Compacting Segments](../design/coordinator.html#compacting-segments). A description of the compaction config is: @@ -853,7 +861,7 @@ If this is the case, the coordinator's automatic compaction might get stuck beca This kind of problem may happen especially in Kafka/Kinesis indexing systems which allow late data arrival. If you see this problem, it's recommended to set `skipOffsetFromLatest` to some large enough value to avoid such conflicts between compaction tasks and realtime tasks. -##### Compaction TuningConfig +###### Compaction TuningConfig |Property|Description|Required| |--------|-----------|--------| @@ -863,15 +871,15 @@ If you see this problem, it's recommended to set `skipOffsetFromLatest` to some |`maxPendingPersists`|See [tuningConfig for indexTask](../ingestion/native_tasks.html#tuningconfig)|no (default = 0 (meaning one persist can be running concurrently with ingestion, and none can be queued up))| |`pushTimeout`|See [tuningConfig for indexTask](../ingestion/native_tasks.html#tuningconfig)|no (default = 0)| -## Overlord +### Overlord -For general Overlord Node information, see [here](../design/indexing-service.html). +For general Overlord Node information, see [here](../design/overlord.html). -### Overlord Static Configuration +#### Overlord Static Configuration -These overlord static configurations can be defined in the `overlord/runtime.properties` file. +These Overlord static configurations can be defined in the `overlord/runtime.properties` file. -#### Overlord Node Configs +##### Overlord Node Configs |Property|Description|Default| |--------|-----------|-------| @@ -881,30 +889,30 @@ These overlord static configurations can be defined in the `overlord/runtime.pro |`druid.tlsPort`|TLS port for HTTPS connector, if [druid.enableTlsPort](../operations/tls-support.html) is set then this config will be used. If `druid.host` contains port then that port will be ignored. This should be a non-negative Integer.|8290| |`druid.service`|The name of the service. This is used as a dimension when emitting metrics and alerts to differentiate between the various services|druid/overlord| -#### Overlord Operations +##### Overlord Operations |Property|Description|Default| |--------|-----------|-------| |`druid.indexer.runner.type`|Choices "local" or "remote". Indicates whether tasks should be run locally or in a distributed environment. Experimental task runner "httpRemote" is also available which is same as "remote" but uses HTTP to interact with Middle Manaters instead of Zookeeper.|local| -|`druid.indexer.storage.type`|Choices are "local" or "metadata". Indicates whether incoming tasks should be stored locally (in heap) or in metadata storage. Storing incoming tasks in metadata storage allows for tasks to be resumed if the overlord should fail.|local| +|`druid.indexer.storage.type`|Choices are "local" or "metadata". Indicates whether incoming tasks should be stored locally (in heap) or in metadata storage. Storing incoming tasks in metadata storage allows for tasks to be resumed if the Overlord should fail.|local| |`druid.indexer.storage.recentlyFinishedThreshold`|A duration of time to store task results.|PT24H| |`druid.indexer.queue.maxSize`|Maximum number of active tasks at one time.|Integer.MAX_VALUE| -|`druid.indexer.queue.startDelay`|Sleep this long before starting overlord queue management. This can be useful to give a cluster time to re-orient itself after e.g. a widespread network issue.|PT1M| -|`druid.indexer.queue.restartDelay`|Sleep this long when overlord queue management throws an exception before trying again.|PT30S| -|`druid.indexer.queue.storageSyncRate`|Sync overlord state this often with an underlying task persistence mechanism.|PT1M| +|`druid.indexer.queue.startDelay`|Sleep this long before starting Overlord queue management. This can be useful to give a cluster time to re-orient itself after e.g. a widespread network issue.|PT1M| +|`druid.indexer.queue.restartDelay`|Sleep this long when Overlord queue management throws an exception before trying again.|PT30S| +|`druid.indexer.queue.storageSyncRate`|Sync Overlord state this often with an underlying task persistence mechanism.|PT1M| -The following configs only apply if the overlord is running in remote mode. For a description of local vs. remote mode, please see (../design/indexing-service.html#overlord-node). +The following configs only apply if the Overlord is running in remote mode. For a description of local vs. remote mode, please see (../design/overlord.html). |Property|Description|Default| |--------|-----------|-------| -|`druid.indexer.runner.taskAssignmentTimeout`|How long to wait after a task as been assigned to a middle manager before throwing an error.|PT5M| -|`druid.indexer.runner.minWorkerVersion`|The minimum middle manager version to send tasks to. |"0"| -|`druid.indexer.runner.compressZnodes`|Indicates whether or not the overlord should expect middle managers to compress Znodes.|true| +|`druid.indexer.runner.taskAssignmentTimeout`|How long to wait after a task as been assigned to a MiddleManager before throwing an error.|PT5M| +|`druid.indexer.runner.minWorkerVersion`|The minimum MiddleManager version to send tasks to. |"0"| +|`druid.indexer.runner.compressZnodes`|Indicates whether or not the Overlord should expect MiddleManagers to compress Znodes.|true| |`druid.indexer.runner.maxZnodeBytes`|The maximum size Znode in bytes that can be created in Zookeeper.|524288| -|`druid.indexer.runner.taskCleanupTimeout`|How long to wait before failing a task after a middle manager is disconnected from Zookeeper.|PT15M| -|`druid.indexer.runner.taskShutdownLinkTimeout`|How long to wait on a shutdown request to a middle manager before timing out|PT1M| +|`druid.indexer.runner.taskCleanupTimeout`|How long to wait before failing a task after a MiddleManager is disconnected from Zookeeper.|PT15M| +|`druid.indexer.runner.taskShutdownLinkTimeout`|How long to wait on a shutdown request to a MiddleManager before timing out|PT1M| |`druid.indexer.runner.pendingTasksRunnerNumThreads`|Number of threads to allocate pending-tasks to workers, must be at least 1.|1| -|`druid.indexer.runner.maxRetriesBeforeBlacklist`|Number of consecutive times the middle manager can fail tasks, before the worker is blacklisted, must be at least 1|5| +|`druid.indexer.runner.maxRetriesBeforeBlacklist`|Number of consecutive times the MiddleManager can fail tasks, before the worker is blacklisted, must be at least 1|5| |`druid.indexer.runner.workerBlackListBackoffTime`|How long to wait before a task is whitelisted again. This value should be greater that the value set for taskBlackListCleanupPeriod.|PT15M| |`druid.indexer.runner.workerBlackListCleanupPeriod`|A duration after which the cleanup thread will startup to clean blacklisted workers.|PT5M| |`druid.indexer.runner.maxPercentageBlacklistWorkers`|The maximum percentage of workers to blacklist, this must be between 0 and 100.|20| @@ -915,21 +923,21 @@ There are additional configs for autoscaling (if it is enabled): |--------|-----------|-------| |`druid.indexer.autoscale.strategy`|Choices are "noop" or "ec2". Sets the strategy to run when autoscaling is required.|noop| |`druid.indexer.autoscale.doAutoscale`|If set to "true" autoscaling will be enabled.|false| -|`druid.indexer.autoscale.provisionPeriod`|How often to check whether or not new middle managers should be added.|PT1M| -|`druid.indexer.autoscale.terminatePeriod`|How often to check when middle managers should be removed.|PT5M| +|`druid.indexer.autoscale.provisionPeriod`|How often to check whether or not new MiddleManagers should be added.|PT1M| +|`druid.indexer.autoscale.terminatePeriod`|How often to check when MiddleManagers should be removed.|PT5M| |`druid.indexer.autoscale.originTime`|The starting reference timestamp that the terminate period increments upon.|2012-01-01T00:55:00.000Z| |`druid.indexer.autoscale.workerIdleTimeout`|How long can a worker be idle (not a run task) before it can be considered for termination.|PT90M| -|`druid.indexer.autoscale.maxScalingDuration`|How long the overlord will wait around for a middle manager to show up before giving up.|PT15M| +|`druid.indexer.autoscale.maxScalingDuration`|How long the Overlord will wait around for a MiddleManager to show up before giving up.|PT15M| |`druid.indexer.autoscale.numEventsToTrack`|The number of autoscaling related events (node creation and termination) to track.|10| -|`druid.indexer.autoscale.pendingTaskTimeout`|How long a task can be in "pending" state before the overlord tries to scale up.|PT30S| +|`druid.indexer.autoscale.pendingTaskTimeout`|How long a task can be in "pending" state before the Overlord tries to scale up.|PT30S| |`druid.indexer.autoscale.workerVersion`|If set, will only create nodes of set version during autoscaling. Overrides dynamic configuration. |null| -|`druid.indexer.autoscale.workerPort`|The port that middle managers will run on.|8080| +|`druid.indexer.autoscale.workerPort`|The port that MiddleManagers will run on.|8080| -### Overlord Dynamic Configuration +#### Overlord Dynamic Configuration -The overlord can dynamically change worker behavior. +The Overlord can dynamically change worker behavior. -The JSON object can be submitted to the overlord via a POST request at: +The JSON object can be submitted to the Overlord via a POST request at: ``` http://:/druid/indexer/v1/worker @@ -984,7 +992,7 @@ Issuing a GET request at the same URL will return the current worker config spec |Property|Description|Default| |--------|-----------|-------| -|`selectStrategy`|How to assign tasks to middle managers. Choices are `fillCapacity`, `equalDistribution`, and `javascript`.|equalDistribution| +|`selectStrategy`|How to assign tasks to MiddleManagers. Choices are `fillCapacity`, `equalDistribution`, and `javascript`.|equalDistribution| |`autoScaler`|Only used if autoscaling is enabled. See below.|null| To view the audit history of worker config issue a GET request to the URL - @@ -993,7 +1001,7 @@ To view the audit history of worker config issue a GET request to the URL - http://:/druid/indexer/v1/worker/history?interval= ``` -default value of interval can be specified by setting `druid.audit.manager.auditHistoryMillis` (1 week if not configured) in overlord runtime.properties. +default value of interval can be specified by setting `druid.audit.manager.auditHistoryMillis` (1 week if not configured) in Overlord runtime.properties. To view last entries of the audit history of worker config issue a GET request to the URL - @@ -1001,11 +1009,11 @@ To view last entries of the audit history of worker config issue a GET reque http://:/druid/indexer/v1/worker/history?count= ``` -#### Worker Select Strategy +##### Worker Select Strategy Worker select strategies control how Druid assigns tasks to middleManagers. -##### Equal Distribution +###### Equal Distribution Tasks are assigned to the middleManager with the most available capacity at the time the task begins running. This is useful if you want work evenly distributed across your middleManagers. @@ -1015,7 +1023,7 @@ useful if you want work evenly distributed across your middleManagers. |`type`|`equalDistribution`.|required; must be `equalDistribution`| |`affinityConfig`|[Affinity config](#affinity) object|null (no affinity)| -##### Fill Capacity +###### Fill Capacity Tasks are assigned to the worker with the most currently-running tasks at the time the task begins running. This is useful in situations where you are elastically auto-scaling middleManagers, since it will tend to pack some full and @@ -1029,7 +1037,7 @@ middleManagers up to capacity simultaneously, rather than a single middleManager |`type`|`fillCapacity`.|required; must be `fillCapacity`| |`affinityConfig`|[Affinity config](#affinity) object|null (no affinity)| -##### Javascript +###### Javascript Allows defining arbitrary logic for selecting workers to run task using a JavaScript function. The function is passed remoteTaskRunnerConfig, map of workerId to available workers and task to be executed and returns the workerId on which the task should be run or null if the task cannot be run. @@ -1055,7 +1063,7 @@ Example: a function that sends batch_index_task to workers 10.0.0.1 and 10.0.0.2 JavaScript-based functionality is disabled by default. Please refer to the Druid JavaScript programming guide for guidelines about using Druid's JavaScript functionality, including instructions on how to enable it. -##### Affinity +###### Affinity Affinity configs can be provided to the _equalDistribution_ and _fillCapacity_ strategies using the "affinityConfig" field. If not provided, the default is to not use affinity at all. @@ -1065,7 +1073,7 @@ field. If not provided, the default is to not use affinity at all. |`affinity`|JSON object mapping a datasource String name to a list of indexing service middleManager host:port String values. Druid doesn't perform DNS resolution, so the 'host' value must match what is configured on the middleManager and what the middleManager announces itself as (examine the Overlord logs to see what your middleManager announces itself as).|{}| |`strong`|With weak affinity (the default), tasks for a dataSource may be assigned to other middleManagers if their affinity-mapped middleManagers are not able to run all pending tasks in the queue for that dataSource. With strong affinity, tasks for a dataSource will only ever be assigned to their affinity-mapped middleManagers, and will wait in the pending queue if necessary.|false| -#### Autoscaler +##### Autoscaler Amazon's EC2 is currently the only supported autoscaler. @@ -1077,11 +1085,15 @@ Amazon's EC2 is currently the only supported autoscaler. |`nodeData`|A JSON object that describes how to launch new nodes.|none; required| |`userData`|A JSON object that describes how to configure new nodes. If you have set druid.indexer.autoscale.workerVersion, this must have a versionReplacementString. Otherwise, a versionReplacementString is not necessary.|none; optional| -## MiddleManager and Peons +## Data Server + +This section contains the configuration options for the processes that reside on Data servers (MiddleManagers/Peons and Historicals) in the suggested [three-server configuration](../design/processes.html#server-types). + +### MiddleManager and Peons These MiddleManager and Peon configurations can be defined in the `middleManager/runtime.properties` file. -### MiddleManager Node Config +#### MiddleManager Node Config |Property|Description|Default| |--------|-----------|-------| @@ -1091,14 +1103,14 @@ These MiddleManager and Peon configurations can be defined in the `middleManager |`druid.tlsPort`|TLS port for HTTPS connector, if [druid.enableTlsPort](../operations/tls-support.html) is set then this config will be used. If `druid.host` contains port then that port will be ignored. This should be a non-negative Integer.|8291| |`druid.service`|The name of the service. This is used as a dimension when emitting metrics and alerts to differentiate between the various services|druid/middlemanager| -### MiddleManager Configuration +#### MiddleManager Configuration -Middle managers pass their configurations down to their child peons. The middle manager requires the following configs: +Middle managers pass their configurations down to their child peons. The MiddleManager requires the following configs: |Property|Description|Default| |--------|-----------|-------| |`druid.indexer.runner.allowedPrefixes`|Whitelist of prefixes for configs that can be passed down to child peons.|"com.metamx", "druid", "org.apache.druid", "user.timezone", "file.encoding", "java.io.tmpdir", "hadoop"| -|`druid.indexer.runner.compressZnodes`|Indicates whether or not the middle managers should compress Znodes.|true| +|`druid.indexer.runner.compressZnodes`|Indicates whether or not the MiddleManagers should compress Znodes.|true| |`druid.indexer.runner.classpath`|Java classpath for the peon.|System.getProperty("java.class.path")| |`druid.indexer.runner.javaCommand`|Command required to execute java.|java| |`druid.indexer.runner.javaOpts`|*DEPRECATED* A string of -X Java options to pass to the peon's JVM. Quotable parameters or parameters with spaces are encouraged to use javaOptsArray|""| @@ -1108,10 +1120,10 @@ Middle managers pass their configurations down to their child peons. The middle |`druid.indexer.runner.endPort`|Ending port used for peon processes, should be greater than or equal to `druid.indexer.runner.startPort` and less than 65536.|65535| |`druid.indexer.runner.ports`|A json array of integers to specify ports that used for peon processes. If provided and non-empty, ports for peon processes will be chosen from these ports. And `druid.indexer.runner.startPort/druid.indexer.runner.endPort` will be completely ignored.|`[]`| |`druid.worker.ip`|The IP of the worker.|localhost| -|`druid.worker.version`|Version identifier for the middle manager.|0| -|`druid.worker.capacity`|Maximum number of tasks the middle manager can accept.|Number of available processors - 1| +|`druid.worker.version`|Version identifier for the MiddleManager.|0| +|`druid.worker.capacity`|Maximum number of tasks the MiddleManager can accept.|Number of available processors - 1| -### Peon Processing +#### Peon Processing Processing properties set on the Middlemanager will be passed through to Peons. @@ -1119,7 +1131,7 @@ Processing properties set on the Middlemanager will be passed through to Peons. |--------|-----------|-------| |`druid.processing.buffer.sizeBytes`|This specifies a buffer size for the storage of intermediate results. The computation engine in both the Historical and Realtime nodes will use a scratch buffer of this size to do all of their intermediate computations off-heap. Larger values allow for more aggregations in a single pass over the data while smaller values can require more passes depending on the query that is being executed.|auto (max 1GB)| |`druid.processing.buffer.poolCacheMaxCount`|processing buffer pool caches the buffers for later use, this is the maximum count cache will grow to. note that pool can create more buffers than it can cache if necessary.|Integer.MAX_VALUE| -|`druid.processing.formatString`|Realtime and historical nodes use this format string to name their processing threads.|processing-%s| +|`druid.processing.formatString`|Realtime and Historical nodes use this format string to name their processing threads.|processing-%s| |`druid.processing.numMergeBuffers`|The number of direct memory buffers available for merging query results. The buffers are sized by `druid.processing.buffer.sizeBytes`. This property is effectively a concurrency limit for queries that require merging buffers. If you are using any queries that require merge buffers (currently, just groupBy v2) then you should have at least two of these.|`max(2, druid.processing.numThreads / 4)`| |`druid.processing.numThreads`|The number of processing threads to have available for parallel processing of segments. Our rule of thumb is `num_cores - 1`, which means that even under heavy load there will still be one core available to do background tasks like talking with ZooKeeper and pulling down segments. If only one core is available, this property defaults to the value `1`.|Number of cores - 1 (or 1)| |`druid.processing.columnCache.sizeBytes`|Maximum size in bytes for the dimension value lookup cache. Any value greater than `0` enables the cache. It is currently disabled by default. Enabling the lookup cache can significantly improve the performance of aggregators operating on dimension values, such as the JavaScript aggregator, or cardinality aggregator, but can slow things down if the cache hit rate is low (i.e. dimensions with few repeating values). Enabling it may also require additional garbage collection tuning to avoid long GC pauses.|`0` (disabled)| @@ -1131,11 +1143,11 @@ The amount of direct memory needed by Druid is at least ensure at least this amount of direct memory is available by providing `-XX:MaxDirectMemorySize=` in `druid.indexer.runner.javaOptsArray` as documented above. -### Peon Query Configuration +#### Peon Query Configuration See [general query configuration](#general-query-configuration). -### Peon Caching +#### Peon Caching You can optionally configure caching to be enabled on the peons by setting caching configs here. @@ -1148,8 +1160,8 @@ You can optionally configure caching to be enabled on the peons by setting cachi See [cache configuration](#cache-configuration) for how to configure cache settings. -### Additional Peon Configuration -Although peons inherit the configurations of their parent middle managers, explicit child peon configs in middle manager can be set by prefixing them with: +#### Additional Peon Configuration +Although peons inherit the configurations of their parent MiddleManagers, explicit child peon configs in MiddleManager can be set by prefixing them with: ``` druid.indexer.fork.property @@ -1169,15 +1181,15 @@ Additional peon configs include: |`druid.indexer.task.restoreTasksOnRestart`|If true, middleManagers will attempt to stop tasks gracefully on shutdown and restore them on restart.|false| |`druid.indexer.server.maxChatRequests`|Maximum number of concurrent requests served by a task's chat handler. Set to 0 to disable limiting.|0| -If the peon is running in remote mode, there must be an overlord up and running. Peons in remote mode can set the following configurations: +If the peon is running in remote mode, there must be an Overlord up and running. Peons in remote mode can set the following configurations: |Property|Description|Default| |--------|-----------|-------| -|`druid.peon.taskActionClient.retry.minWait`|The minimum retry time to communicate with overlord.|PT5S| -|`druid.peon.taskActionClient.retry.maxWait`|The maximum retry time to communicate with overlord.|PT1M| -|`druid.peon.taskActionClient.retry.maxRetryCount`|The maximum number of retries to communicate with overlord.|60| +|`druid.peon.taskActionClient.retry.minWait`|The minimum retry time to communicate with Overlord.|PT5S| +|`druid.peon.taskActionClient.retry.maxWait`|The maximum retry time to communicate with Overlord.|PT1M| +|`druid.peon.taskActionClient.retry.maxRetryCount`|The maximum number of retries to communicate with Overlord.|60| -#### SegmentWriteOutMediumFactory +##### SegmentWriteOutMediumFactory When new segments are created, Druid temporarily stores some pre-processed data in some buffers. Currently two types of *medium* exist for those buffers: *temporary files* and *off-heap memory*. @@ -1200,33 +1212,47 @@ then the value from the configuration below is used: |--------|-----------|-------| |`druid.peon.defaultSegmentWriteOutMediumFactory.type`|`tmpFile` or `offHeapMemory`, see explanation above|`tmpFile`| -## Broker +### Historical -For general Broker Node information, see [here](../design/broker.html). +For general Historical Node information, see [here](../design/historical.html). -These Broker configurations can be defined in the `broker/runtime.properties` file. - -### Broker Node Configs +These Historical configurations can be defined in the `historical/runtime.properties` file. +#### Historical Node Configuration |Property|Description|Default| |--------|-----------|-------| |`druid.host`|The host for the current node. This is used to advertise the current processes location as reachable from another node and should generally be specified such that `http://${druid.host}/` could actually talk to this process|InetAddress.getLocalHost().getCanonicalHostName()| |`druid.bindOnHost`|Indicating whether the node's internal jetty server bind on `druid.host`. Default is false, which means binding to all interfaces.|false| -|`druid.plaintextPort`|This is the port to actually listen on; unless port mapping is used, this will be the same port as is on `druid.host`|8082| -|`druid.tlsPort`|TLS port for HTTPS connector, if [druid.enableTlsPort](../operations/tls-support.html) is set then this config will be used. If `druid.host` contains port then that port will be ignored. This should be a non-negative Integer.|8282| -|`druid.service`|The name of the service. This is used as a dimension when emitting metrics and alerts to differentiate between the various services|druid/broker| +|`druid.plaintextPort`|This is the port to actually listen on; unless port mapping is used, this will be the same port as is on `druid.host`|8083| +|`druid.tlsPort`|TLS port for HTTPS connector, if [druid.enableTlsPort](../operations/tls-support.html) is set then this config will be used. If `druid.host` contains port then that port will be ignored. This should be a non-negative Integer.|8283| +|`druid.service`|The name of the service. This is used as a dimension when emitting metrics and alerts to differentiate between the various services|druid/historical| -### Query Configuration -#### Query Prioritization +#### Historical General Configuration -|Property|Possible Values|Description|Default| -|--------|---------------|-----------|-------| -|`druid.broker.balancer.type`|`random`, `connectionCount`|Determines how the broker balances connections to historical nodes. `random` choose randomly, `connectionCount` picks the node with the fewest number of active connections to|`random`| -|`druid.broker.select.tier`|`highestPriority`, `lowestPriority`, `custom`|If segments are cross-replicated across tiers in a cluster, you can tell the broker to prefer to select segments in a tier with a certain priority.|`highestPriority`| -|`druid.broker.select.tier.custom.priorities`|`An array of integer priorities.`|Select servers in tiers with a custom priority list.|None| +|Property|Description|Default| +|--------|-----------|-------| +|`druid.server.maxSize`|The maximum number of bytes-worth of segments that the node wants assigned to it. This is not a limit that Historical nodes actually enforces, just a value published to the Coordinator node so it can plan accordingly.|0| +|`druid.server.tier`| A string to name the distribution tier that the storage node belongs to. Many of the [rules Coordinator nodes use](../operations/rule-configuration.html) to manage segments can be keyed on tiers. | `_default_tier` | +|`druid.server.priority`|In a tiered architecture, the priority of the tier, thus allowing control over which nodes are queried. Higher numbers mean higher priority. The default (no priority) works for architecture with no cross replication (tiers that have no data-storage overlap). Data centers typically have equal priority. | 0 | -#### Server Configuration +#### Storing Segments + +|Property|Description|Default| +|--------|-----------|-------| +|`druid.segmentCache.locations`|Segments assigned to a Historical node are first stored on the local file system (in a disk cache) and then served by the Historical node. These locations define where that local cache resides. This value cannot be NULL or EMPTY. Here is an example `druid.segmentCache.locations=[{"path": "/mnt/druidSegments", "maxSize": 10000, "freeSpacePercent": 1.0}]`. "freeSpacePercent" is optional, if provided then enforces that much of free disk partition space while storing segments. But, it depends on File.getTotalSpace() and File.getFreeSpace() methods, so enable if only if they work for your File System.| none | +|`druid.segmentCache.deleteOnRemove`|Delete segment files from cache once a node is no longer serving a segment.|true| +|`druid.segmentCache.dropSegmentDelayMillis`|How long a node delays before completely dropping segment.|30000 (30 seconds)| +|`druid.segmentCache.infoDir`|Historical nodes keep track of the segments they are serving so that when the process is restarted they can reload the same segments without waiting for the Coordinator to reassign. This path defines where this metadata is kept. Directory will be created if needed.|${first_location}/info_dir| +|`druid.segmentCache.announceIntervalMillis`|How frequently to announce segments while segments are loading from cache. Set this value to zero to wait for all segments to be loaded before announcing.|5000 (5 seconds)| +|`druid.segmentCache.numLoadingThreads`|How many segments to drop or load concurrently from from deep storage.|10| +|`druid.segmentCache.numBootstrapThreads`|How many segments to load concurrently from local storage at startup.|Same as numLoadingThreads| + +In `druid.segmentCache.locations`, *freeSpacePercent* was added because *maxSize* setting is only a theoretical limit and assumes that much space will always be available for storing segments. In case of any druid bug leading to unaccounted segment files left alone on disk or some other process writing stuff to disk, This check can start failing segment loading early before filling up the disk completely and leaving the host usable otherwise. + +#### Historical Query Configs + +##### Concurrent Requests Druid uses Jetty to serve HTTP requests. @@ -1237,42 +1263,18 @@ Druid uses Jetty to serve HTTP requests. |`druid.server.http.maxIdleTime`|The Jetty max idle time for a connection.|PT5M| |`druid.server.http.enableRequestLimit`|If enabled, no requests would be queued in jetty queue and "HTTP 429 Too Many Requests" error response would be sent. |false| |`druid.server.http.defaultQueryTimeout`|Query timeout in millis, beyond which unfinished queries will be cancelled|300000| -|`druid.server.http.maxScatterGatherBytes`|Maximum number of bytes gathered from data nodes such as historicals and realtime processes to execute a query. Queries that exceed this limit will fail. This is an advance configuration that allows to protect in case broker is under heavy load and not utilizing the data gathered in memory fast enough and leading to OOMs. This limit can be further reduced at query time using `maxScatterGatherBytes` in the context. Note that having large limit is not necessarily bad if broker is never under heavy concurrent load in which case data gathered is processed quickly and freeing up the memory used.|Long.MAX_VALUE| |`druid.server.http.gracefulShutdownTimeout`|The maximum amount of time Jetty waits after receiving shutdown signal. After this timeout the threads will be forcefully shutdown. This allows any queries that are executing to complete.|`PT0S` (do not wait)| |`druid.server.http.unannouncePropagationDelay`|How long to wait for zookeeper unannouncements to propagate before shutting down Jetty. This is a minimum and `druid.server.http.gracefulShutdownTimeout` does not start counting down until after this period elapses.|`PT0S` (do not wait)| |`druid.server.http.maxQueryTimeout`|Maximum allowed value (in milliseconds) for `timeout` parameter. See [query-context](../querying/query-context.html) to know more about `timeout`. Query is rejected if the query context `timeout` is greater than this value. |Long.MAX_VALUE| -|`druid.server.http.maxRequestHeaderSize`|Maximum size of a request header in bytes. Larger headers consume more memory and can make a server more vulnerable to denial of service attacks. |8 * 1024| +|`druid.server.http.maxRequestHeaderSize`|Maximum size of a request header in bytes. Larger headers consume more memory and can make a server more vulnerable to denial of service attacks.|8 * 1024| -#### Client Configuration - -Druid Brokers use an HTTP client to communicate with with data servers (historical servers and real-time tasks). This -client has the following configuration options. - -|Property|Description|Default| -|--------|-----------|-------| -|`druid.broker.http.numConnections`|Size of connection pool for the Broker to connect to historical and real-time processes. If there are more queries than this number that all need to speak to the same node, then they will queue up.|20| -|`druid.broker.http.compressionCodec`|Compression codec the Broker uses to communicate with historical and real-time processes. May be "gzip" or "identity".|gzip| -|`druid.broker.http.readTimeout`|The timeout for data reads from historical servers and real-time tasks.|PT15M| -|`druid.broker.http.unusedConnectionTimeout`|The timeout for idle connections in connection pool. This timeout should be less than `druid.broker.http.readTimeout`. Set this timeout = ~90% of `druid.broker.http.readTimeout`|`PT4M`| -|`druid.broker.http.maxQueuedBytes`|Maximum number of bytes queued per query before exerting backpressure on the channel to the data server. Similar to `druid.server.http.maxScatterGatherBytes`, except unlike that configuration, this one will trigger backpressure rather than query failure. Zero means disabled. Can be overridden by the ["maxQueuedBytes" query context parameter](../querying/query-context.html).|0 (disabled)| - -#### Retry Policy - -Druid broker can optionally retry queries internally for transient errors. - -|Property|Description|Default| -|--------|-----------|-------| -|`druid.broker.retryPolicy.numTries`|Number of tries.|1| - -#### Processing - -The broker uses processing configs for nested groupBy queries. And, if you use groupBy v1, long-interval queries (of any type) can be broken into shorter interval queries and processed in parallel inside this thread pool. For more details, see "chunkPeriod" in [Query Context](../querying/query-context.html) doc. +##### Processing |Property|Description|Default| |--------|-----------|-------| |`druid.processing.buffer.sizeBytes`|This specifies a buffer size for the storage of intermediate results. The computation engine in both the Historical and Realtime nodes will use a scratch buffer of this size to do all of their intermediate computations off-heap. Larger values allow for more aggregations in a single pass over the data while smaller values can require more passes depending on the query that is being executed.|auto (max 1GB)| |`druid.processing.buffer.poolCacheMaxCount`|processing buffer pool caches the buffers for later use, this is the maximum count cache will grow to. note that pool can create more buffers than it can cache if necessary.|Integer.MAX_VALUE| -|`druid.processing.formatString`|Realtime and historical nodes use this format string to name their processing threads.|processing-%s| +|`druid.processing.formatString`|Realtime and Historical nodes use this format string to name their processing threads.|processing-%s| |`druid.processing.numMergeBuffers`|The number of direct memory buffers available for merging query results. The buffers are sized by `druid.processing.buffer.sizeBytes`. This property is effectively a concurrency limit for queries that require merging buffers. If you are using any queries that require merge buffers (currently, just groupBy v2) then you should have at least two of these.|`max(2, druid.processing.numThreads / 4)`| |`druid.processing.numThreads`|The number of processing threads to have available for parallel processing of segments. Our rule of thumb is `num_cores - 1`, which means that even under heavy load there will still be one core available to do background tasks like talking with ZooKeeper and pulling down segments. If only one core is available, this property defaults to the value `1`.|Number of cores - 1 (or 1)| |`druid.processing.columnCache.sizeBytes`|Maximum size in bytes for the dimension value lookup cache. Any value greater than `0` enables the cache. It is currently disabled by default. Enabling the lookup cache can significantly improve the performance of aggregators operating on dimension values, such as the JavaScript aggregator, or cardinality aggregator, but can slow things down if the cache hit rate is low (i.e. dimensions with few repeating values). Enabling it may also require additional garbage collection tuning to avoid long GC pauses.|`0` (disabled)| @@ -1284,13 +1286,117 @@ The amount of direct memory needed by Druid is at least ensure at least this amount of direct memory is available by providing `-XX:MaxDirectMemorySize=` at the command line. -#### Broker Query Configuration +##### Historical Query Configuration See [general query configuration](#general-query-configuration). -### SQL +#### Historical Caching -The Druid SQL server is configured through the following properties on the broker. +You can optionally only configure caching to be enabled on the Historical by setting caching configs here. + +|Property|Possible Values|Description|Default| +|--------|---------------|-----------|-------| +|`druid.historical.cache.useCache`|true, false|Enable the cache on the Historical.|false| +|`druid.historical.cache.populateCache`|true, false|Populate the cache on the Historical.|false| +|`druid.historical.cache.unCacheable`|All druid query types|All query types to not cache.|["groupBy", "select"]| + +See [cache configuration](#cache-configuration) for how to configure cache settings. + +## Query Server + +This section contains the configuration options for the processes that reside on Query servers (Brokers) in the suggested [three-server configuration](../design/processes.html#server-types). + +### Broker + +For general Broker process information, see [here](../design/broker.html). + +These Broker configurations can be defined in the `broker/runtime.properties` file. + +#### Broker Node Configs + +|Property|Description|Default| +|--------|-----------|-------| +|`druid.host`|The host for the current node. This is used to advertise the current processes location as reachable from another node and should generally be specified such that `http://${druid.host}/` could actually talk to this process|InetAddress.getLocalHost().getCanonicalHostName()| +|`druid.bindOnHost`|Indicating whether the node's internal jetty server bind on `druid.host`. Default is false, which means binding to all interfaces.|false| +|`druid.plaintextPort`|This is the port to actually listen on; unless port mapping is used, this will be the same port as is on `druid.host`|8082| +|`druid.tlsPort`|TLS port for HTTPS connector, if [druid.enableTlsPort](../operations/tls-support.html) is set then this config will be used. If `druid.host` contains port then that port will be ignored. This should be a non-negative Integer.|8282| +|`druid.service`|The name of the service. This is used as a dimension when emitting metrics and alerts to differentiate between the various services|druid/broker| + +#### Query Configuration + +##### Query Prioritization + +|Property|Possible Values|Description|Default| +|--------|---------------|-----------|-------| +|`druid.broker.balancer.type`|`random`, `connectionCount`|Determines how the broker balances connections to Historical nodes. `random` choose randomly, `connectionCount` picks the node with the fewest number of active connections to|`random`| +|`druid.broker.select.tier`|`highestPriority`, `lowestPriority`, `custom`|If segments are cross-replicated across tiers in a cluster, you can tell the broker to prefer to select segments in a tier with a certain priority.|`highestPriority`| +|`druid.broker.select.tier.custom.priorities`|`An array of integer priorities.`|Select servers in tiers with a custom priority list.|None| + +##### Server Configuration + +Druid uses Jetty to serve HTTP requests. + +|Property|Description|Default| +|--------|-----------|-------| +|`druid.server.http.numThreads`|Number of threads for HTTP requests.|max(10, (Number of cores * 17) / 16 + 2) + 30| +|`druid.server.http.queueSize`|Size of the worker queue used by Jetty server to temporarily store incoming client connections. If this value is set and a request is rejected by jetty because queue is full then client would observe request failure with TCP connection being closed immediately with a completely empty response from server.|Unbounded| +|`druid.server.http.maxIdleTime`|The Jetty max idle time for a connection.|PT5M| +|`druid.server.http.enableRequestLimit`|If enabled, no requests would be queued in jetty queue and "HTTP 429 Too Many Requests" error response would be sent. |false| +|`druid.server.http.defaultQueryTimeout`|Query timeout in millis, beyond which unfinished queries will be cancelled|300000| +|`druid.server.http.maxScatterGatherBytes`|Maximum number of bytes gathered from data nodes such as Historicals and realtime processes to execute a query. Queries that exceed this limit will fail. This is an advance configuration that allows to protect in case Broker is under heavy load and not utilizing the data gathered in memory fast enough and leading to OOMs. This limit can be further reduced at query time using `maxScatterGatherBytes` in the context. Note that having large limit is not necessarily bad if broker is never under heavy concurrent load in which case data gathered is processed quickly and freeing up the memory used.|Long.MAX_VALUE| +|`druid.server.http.gracefulShutdownTimeout`|The maximum amount of time Jetty waits after receiving shutdown signal. After this timeout the threads will be forcefully shutdown. This allows any queries that are executing to complete.|`PT0S` (do not wait)| +|`druid.server.http.unannouncePropagationDelay`|How long to wait for zookeeper unannouncements to propagate before shutting down Jetty. This is a minimum and `druid.server.http.gracefulShutdownTimeout` does not start counting down until after this period elapses.|`PT0S` (do not wait)| +|`druid.server.http.maxQueryTimeout`|Maximum allowed value (in milliseconds) for `timeout` parameter. See [query-context](../querying/query-context.html) to know more about `timeout`. Query is rejected if the query context `timeout` is greater than this value. |Long.MAX_VALUE| +|`druid.server.http.maxRequestHeaderSize`|Maximum size of a request header in bytes. Larger headers consume more memory and can make a server more vulnerable to denial of service attacks. |8 * 1024| + +##### Client Configuration + +Druid Brokers use an HTTP client to communicate with with data servers (Historical servers and real-time tasks). This +client has the following configuration options. + +|Property|Description|Default| +|--------|-----------|-------| +|`druid.broker.http.numConnections`|Size of connection pool for the Broker to connect to Historical and real-time processes. If there are more queries than this number that all need to speak to the same node, then they will queue up.|20| +|`druid.broker.http.compressionCodec`|Compression codec the Broker uses to communicate with Historical and real-time processes. May be "gzip" or "identity".|gzip| +|`druid.broker.http.readTimeout`|The timeout for data reads from Historical servers and real-time tasks.|PT15M| +|`druid.broker.http.unusedConnectionTimeout`|The timeout for idle connections in connection pool. This timeout should be less than `druid.broker.http.readTimeout`. Set this timeout = ~90% of `druid.broker.http.readTimeout`|`PT4M`| +|`druid.broker.http.maxQueuedBytes`|Maximum number of bytes queued per query before exerting backpressure on the channel to the data server. Similar to `druid.server.http.maxScatterGatherBytes`, except unlike that configuration, this one will trigger backpressure rather than query failure. Zero means disabled. Can be overridden by the ["maxQueuedBytes" query context parameter](../querying/query-context.html).|0 (disabled)| + +##### Retry Policy + +Druid broker can optionally retry queries internally for transient errors. + +|Property|Description|Default| +|--------|-----------|-------| +|`druid.broker.retryPolicy.numTries`|Number of tries.|1| + +##### Processing + +The broker uses processing configs for nested groupBy queries. And, if you use groupBy v1, long-interval queries (of any type) can be broken into shorter interval queries and processed in parallel inside this thread pool. For more details, see "chunkPeriod" in [Query Context](../querying/query-context.html) doc. + +|Property|Description|Default| +|--------|-----------|-------| +|`druid.processing.buffer.sizeBytes`|This specifies a buffer size for the storage of intermediate results. The computation engine in both the Historical and Realtime nodes will use a scratch buffer of this size to do all of their intermediate computations off-heap. Larger values allow for more aggregations in a single pass over the data while smaller values can require more passes depending on the query that is being executed.|auto (max 1GB)| +|`druid.processing.buffer.poolCacheMaxCount`|processing buffer pool caches the buffers for later use, this is the maximum count cache will grow to. note that pool can create more buffers than it can cache if necessary.|Integer.MAX_VALUE| +|`druid.processing.formatString`|Realtime and Historical nodes use this format string to name their processing threads.|processing-%s| +|`druid.processing.numMergeBuffers`|The number of direct memory buffers available for merging query results. The buffers are sized by `druid.processing.buffer.sizeBytes`. This property is effectively a concurrency limit for queries that require merging buffers. If you are using any queries that require merge buffers (currently, just groupBy v2) then you should have at least two of these.|`max(2, druid.processing.numThreads / 4)`| +|`druid.processing.numThreads`|The number of processing threads to have available for parallel processing of segments. Our rule of thumb is `num_cores - 1`, which means that even under heavy load there will still be one core available to do background tasks like talking with ZooKeeper and pulling down segments. If only one core is available, this property defaults to the value `1`.|Number of cores - 1 (or 1)| +|`druid.processing.columnCache.sizeBytes`|Maximum size in bytes for the dimension value lookup cache. Any value greater than `0` enables the cache. It is currently disabled by default. Enabling the lookup cache can significantly improve the performance of aggregators operating on dimension values, such as the JavaScript aggregator, or cardinality aggregator, but can slow things down if the cache hit rate is low (i.e. dimensions with few repeating values). Enabling it may also require additional garbage collection tuning to avoid long GC pauses.|`0` (disabled)| +|`druid.processing.fifo`|If the processing queue should treat tasks of equal priority in a FIFO manner|`false`| +|`druid.processing.tmpDir`|Path where temporary files created while processing a query should be stored. If specified, this configuration takes priority over the default `java.io.tmpdir` path.|path represented by `java.io.tmpdir`| + +The amount of direct memory needed by Druid is at least +`druid.processing.buffer.sizeBytes * (druid.processing.numMergeBuffers + druid.processing.numThreads + 1)`. You can +ensure at least this amount of direct memory is available by providing `-XX:MaxDirectMemorySize=` at the command +line. + +##### Broker Query Configuration + +See [general query configuration](#general-query-configuration). + +#### SQL + +The Druid SQL server is configured through the following properties on the Broker. |Property|Description|Default| |--------|-----------|-------| @@ -1309,136 +1415,46 @@ The Druid SQL server is configured through the following properties on the broke |`druid.sql.planner.selectThreshold`|Page size threshold for [Select queries](../querying/select-query.html). Select queries for larger resultsets will be issued back-to-back using pagination.|1000| |`druid.sql.planner.useApproximateCountDistinct`|Whether to use an approximate cardinalty algorithm for `COUNT(DISTINCT foo)`.|true| |`druid.sql.planner.useApproximateTopN`|Whether to use approximate [TopN queries](../querying/topnquery.html) when a SQL query could be expressed as such. If false, exact [GroupBy queries](../querying/groupbyquery.html) will be used instead.|true| -|`druid.sql.planner.useFallback`|Whether to evaluate operations on the broker when they cannot be expressed as Druid queries. This option is not recommended for production since it can generate unscalable query plans. If false, SQL queries that cannot be translated to Druid queries will fail.|false| +|`druid.sql.planner.useFallback`|Whether to evaluate operations on the Broker when they cannot be expressed as Druid queries. This option is not recommended for production since it can generate unscalable query plans. If false, SQL queries that cannot be translated to Druid queries will fail.|false| |`druid.sql.planner.requireTimeCondition`|Whether to require SQL to have filter conditions on __time column so that all generated native queries will have user specified intervals. If true, all queries wihout filter condition on __time column will fail|false| |`druid.sql.planner.sqlTimeZone`|Sets the default time zone for the server, which will affect how time functions and timestamp literals behave. Should be a time zone name like "America/Los_Angeles" or offset like "-08:00".|UTC| -### Broker Caching +#### Broker Caching -You can optionally only configure caching to be enabled on the broker by setting caching configs here. +You can optionally only configure caching to be enabled on the Broker by setting caching configs here. |Property|Possible Values|Description|Default| |--------|---------------|-----------|-------| -|`druid.broker.cache.useCache`|true, false|Enable the cache on the broker.|false| -|`druid.broker.cache.populateCache`|true, false|Populate the cache on the broker.|false| -|`druid.broker.cache.useResultLevelCache`|true, false|Enable result level caching on the broker.|false| -|`druid.broker.cache.populateResultLevelCache`|true, false|Populate the result level cache on the broker.|false| +|`druid.broker.cache.useCache`|true, false|Enable the cache on the Broker.|false| +|`druid.broker.cache.populateCache`|true, false|Populate the cache on the Broker.|false| +|`druid.broker.cache.useResultLevelCache`|true, false|Enable result level caching on the Broker.|false| +|`druid.broker.cache.populateResultLevelCache`|true, false|Populate the result level cache on the Broker.|false| |`druid.broker.cache.resultLevelCacheLimit`|positive integer|Maximum size of query response that can be cached.|`Integer.MAX_VALUE`| |`druid.broker.cache.unCacheable`|All druid query types|All query types to not cache.|`["groupBy", "select"]`| -|`druid.broker.cache.cacheBulkMergeLimit`|positive integer or 0|Queries with more segments than this number will not attempt to fetch from cache at the broker level, leaving potential caching fetches (and cache result merging) to the historicals|`Integer.MAX_VALUE`| +|`druid.broker.cache.cacheBulkMergeLimit`|positive integer or 0|Queries with more segments than this number will not attempt to fetch from cache at the broker level, leaving potential caching fetches (and cache result merging) to the Historicals|`Integer.MAX_VALUE`| See [cache configuration](#cache-configuration) for how to configure cache settings. -### Segment Discovery +#### Segment Discovery |Property|Possible Values|Description|Default| |--------|---------------|-----------|-------| |`druid.serverview.type`|batch or http|Segment discovery method to use. "http" enables discovering segments using HTTP instead of zookeeper.|batch| -|`druid.broker.segment.watchedTiers`|List of strings|Broker watches the segment announcements from nodes serving segments to build cache of which node is serving which segments, this configuration allows to only consider segments being served from a whitelist of tiers. By default, Broker would consider all tiers. This can be used to partition your dataSources in specific historical tiers and configure brokers in partitions so that they are only queryable for specific dataSources.|none| +|`druid.broker.segment.watchedTiers`|List of strings|Broker watches the segment announcements from nodes serving segments to build cache of which node is serving which segments, this configuration allows to only consider segments being served from a whitelist of tiers. By default, Broker would consider all tiers. This can be used to partition your dataSources in specific Historical tiers and configure brokers in partitions so that they are only queryable for specific dataSources.|none| |`druid.broker.segment.watchedDataSources`|List of strings|Broker watches the segment announcements from nodes serving segments to build cache of which node is serving which segments, this configuration allows to only consider segments being served from a whitelist of dataSources. By default, Broker would consider all datasources. This can be used to configure brokers in partitions so that they are only queryable for specific dataSources.|none| |`druid.broker.segment.awaitInitializationOnStart`|Boolean|Whether the the Broker will wait for its view of segments to fully initialize before starting up. If set to 'true', the Broker's HTTP server will not start up, and the Broker will not announce itself as available, until the server view is initialized. See also `druid.sql.planner.awaitInitializationOnStart`, a related setting.|true| -## Historical - -For general Historical Node information, see [here](../design/historical.html). - -These Historical configurations can be defined in the `historical/runtime.properties` file. - -### Historical Node Configuration -|Property|Description|Default| -|--------|-----------|-------| -|`druid.host`|The host for the current node. This is used to advertise the current processes location as reachable from another node and should generally be specified such that `http://${druid.host}/` could actually talk to this process|InetAddress.getLocalHost().getCanonicalHostName()| -|`druid.bindOnHost`|Indicating whether the node's internal jetty server bind on `druid.host`. Default is false, which means binding to all interfaces.|false| -|`druid.plaintextPort`|This is the port to actually listen on; unless port mapping is used, this will be the same port as is on `druid.host`|8083| -|`druid.tlsPort`|TLS port for HTTPS connector, if [druid.enableTlsPort](../operations/tls-support.html) is set then this config will be used. If `druid.host` contains port then that port will be ignored. This should be a non-negative Integer.|8283| -|`druid.service`|The name of the service. This is used as a dimension when emitting metrics and alerts to differentiate between the various services|druid/historical| - - -### Historical General Configuration - -|Property|Description|Default| -|--------|-----------|-------| -|`druid.server.maxSize`|The maximum number of bytes-worth of segments that the node wants assigned to it. This is not a limit that Historical nodes actually enforces, just a value published to the Coordinator node so it can plan accordingly.|0| -|`druid.server.tier`| A string to name the distribution tier that the storage node belongs to. Many of the [rules Coordinator nodes use](../operations/rule-configuration.html) to manage segments can be keyed on tiers. | `_default_tier` | -|`druid.server.priority`|In a tiered architecture, the priority of the tier, thus allowing control over which nodes are queried. Higher numbers mean higher priority. The default (no priority) works for architecture with no cross replication (tiers that have no data-storage overlap). Data centers typically have equal priority. | 0 | - -### Storing Segments - -|Property|Description|Default| -|--------|-----------|-------| -|`druid.segmentCache.locations`|Segments assigned to a Historical node are first stored on the local file system (in a disk cache) and then served by the Historical node. These locations define where that local cache resides. This value cannot be NULL or EMPTY. Here is an example `druid.segmentCache.locations=[{"path": "/mnt/druidSegments", "maxSize": 10000, "freeSpacePercent": 1.0}]`. "freeSpacePercent" is optional, if provided then enforces that much of free disk partition space while storing segments. But, it depends on File.getTotalSpace() and File.getFreeSpace() methods, so enable if only if they work for your File System.| none | -|`druid.segmentCache.deleteOnRemove`|Delete segment files from cache once a node is no longer serving a segment.|true| -|`druid.segmentCache.dropSegmentDelayMillis`|How long a node delays before completely dropping segment.|30000 (30 seconds)| -|`druid.segmentCache.infoDir`|Historical nodes keep track of the segments they are serving so that when the process is restarted they can reload the same segments without waiting for the Coordinator to reassign. This path defines where this metadata is kept. Directory will be created if needed.|${first_location}/info_dir| -|`druid.segmentCache.announceIntervalMillis`|How frequently to announce segments while segments are loading from cache. Set this value to zero to wait for all segments to be loaded before announcing.|5000 (5 seconds)| -|`druid.segmentCache.numLoadingThreads`|How many segments to drop or load concurrently from from deep storage.|10| -|`druid.segmentCache.numBootstrapThreads`|How many segments to load concurrently from local storage at startup.|Same as numLoadingThreads| - -In `druid.segmentCache.locations`, *freeSpacePercent* was added because *maxSize* setting is only a theoretical limit and assumes that much space will always be available for storing segments. In case of any druid bug leading to unaccounted segment files left alone on disk or some other process writing stuff to disk, This check can start failing segment loading early before filling up the disk completely and leaving the host usable otherwise. - -### Historical Query Configs - -#### Concurrent Requests - -Druid uses Jetty to serve HTTP requests. - -|Property|Description|Default| -|--------|-----------|-------| -|`druid.server.http.numThreads`|Number of threads for HTTP requests.|max(10, (Number of cores * 17) / 16 + 2) + 30| -|`druid.server.http.queueSize`|Size of the worker queue used by Jetty server to temporarily store incoming client connections. If this value is set and a request is rejected by jetty because queue is full then client would observe request failure with TCP connection being closed immediately with a completely empty response from server.|Unbounded| -|`druid.server.http.maxIdleTime`|The Jetty max idle time for a connection.|PT5M| -|`druid.server.http.enableRequestLimit`|If enabled, no requests would be queued in jetty queue and "HTTP 429 Too Many Requests" error response would be sent. |false| -|`druid.server.http.defaultQueryTimeout`|Query timeout in millis, beyond which unfinished queries will be cancelled|300000| -|`druid.server.http.gracefulShutdownTimeout`|The maximum amount of time Jetty waits after receiving shutdown signal. After this timeout the threads will be forcefully shutdown. This allows any queries that are executing to complete.|`PT0S` (do not wait)| -|`druid.server.http.unannouncePropagationDelay`|How long to wait for zookeeper unannouncements to propagate before shutting down Jetty. This is a minimum and `druid.server.http.gracefulShutdownTimeout` does not start counting down until after this period elapses.|`PT0S` (do not wait)| -|`druid.server.http.maxQueryTimeout`|Maximum allowed value (in milliseconds) for `timeout` parameter. See [query-context](../querying/query-context.html) to know more about `timeout`. Query is rejected if the query context `timeout` is greater than this value. |Long.MAX_VALUE| -|`druid.server.http.maxRequestHeaderSize`|Maximum size of a request header in bytes. Larger headers consume more memory and can make a server more vulnerable to denial of service attacks.|8 * 1024| - -#### Processing - -|Property|Description|Default| -|--------|-----------|-------| -|`druid.processing.buffer.sizeBytes`|This specifies a buffer size for the storage of intermediate results. The computation engine in both the Historical and Realtime nodes will use a scratch buffer of this size to do all of their intermediate computations off-heap. Larger values allow for more aggregations in a single pass over the data while smaller values can require more passes depending on the query that is being executed.|auto (max 1GB)| -|`druid.processing.buffer.poolCacheMaxCount`|processing buffer pool caches the buffers for later use, this is the maximum count cache will grow to. note that pool can create more buffers than it can cache if necessary.|Integer.MAX_VALUE| -|`druid.processing.formatString`|Realtime and historical nodes use this format string to name their processing threads.|processing-%s| -|`druid.processing.numMergeBuffers`|The number of direct memory buffers available for merging query results. The buffers are sized by `druid.processing.buffer.sizeBytes`. This property is effectively a concurrency limit for queries that require merging buffers. If you are using any queries that require merge buffers (currently, just groupBy v2) then you should have at least two of these.|`max(2, druid.processing.numThreads / 4)`| -|`druid.processing.numThreads`|The number of processing threads to have available for parallel processing of segments. Our rule of thumb is `num_cores - 1`, which means that even under heavy load there will still be one core available to do background tasks like talking with ZooKeeper and pulling down segments. If only one core is available, this property defaults to the value `1`.|Number of cores - 1 (or 1)| -|`druid.processing.columnCache.sizeBytes`|Maximum size in bytes for the dimension value lookup cache. Any value greater than `0` enables the cache. It is currently disabled by default. Enabling the lookup cache can significantly improve the performance of aggregators operating on dimension values, such as the JavaScript aggregator, or cardinality aggregator, but can slow things down if the cache hit rate is low (i.e. dimensions with few repeating values). Enabling it may also require additional garbage collection tuning to avoid long GC pauses.|`0` (disabled)| -|`druid.processing.fifo`|If the processing queue should treat tasks of equal priority in a FIFO manner|`false`| -|`druid.processing.tmpDir`|Path where temporary files created while processing a query should be stored. If specified, this configuration takes priority over the default `java.io.tmpdir` path.|path represented by `java.io.tmpdir`| - -The amount of direct memory needed by Druid is at least -`druid.processing.buffer.sizeBytes * (druid.processing.numMergeBuffers + druid.processing.numThreads + 1)`. You can -ensure at least this amount of direct memory is available by providing `-XX:MaxDirectMemorySize=` at the command -line. - -#### Historical Query Configuration - -See [general query configuration](#general-query-configuration). - -### Historical Caching - -You can optionally only configure caching to be enabled on the historical by setting caching configs here. - -|Property|Possible Values|Description|Default| -|--------|---------------|-----------|-------| -|`druid.historical.cache.useCache`|true, false|Enable the cache on the historical.|false| -|`druid.historical.cache.populateCache`|true, false|Populate the cache on the historical.|false| -|`druid.historical.cache.unCacheable`|All druid query types|All query types to not cache.|["groupBy", "select"]| - -See [cache configuration](#cache-configuration) for how to configure cache settings. - ## Cache Configuration -This section describes caching configuration that is common to broker, historical, and middleManager/peon nodes. +This section describes caching configuration that is common to Broker, Historical, and MiddleManager/Peon processes. -Caching can optionally be enabled on the broker, historical, and middleManager/peon nodes. See [broker](#broker-caching), -[historical](#historical-caching), and [peon](#peon-caching) configuration options for how to enable it for different processes. +Caching can optionally be enabled on the Broker, Historical, and MiddleManager/Peon processses. See [Broker](#broker-caching), +[Historical](#Historical-caching), and [Peon](#peon-caching) configuration options for how to enable it for different processes. Druid uses a local in-memory cache by default, unless a diffrent type of cache is specified. Use the `druid.cache.type` configuration to set a different kind of cache. Cache settings are set globally, so the same configuration can be re-used -for both broker and historical nodes, when defined in the common properties file. +for both Broker and Historical nodes, when defined in the common properties file. ### Cache Type @@ -1527,12 +1543,12 @@ If there is an L1 miss and L2 hit, it will also populate L1. |`druid.cache.l2.type`|type of cache to use for L2 cache. See `druid.cache.type` configuration for valid types.|`caffeine`| |`druid.cache.l1.*`|Any property valid for the given type of L1 cache can be set using this prefix. For instance, if you are using a `caffeine` L1 cache, specify `druid.cache.l1.sizeInBytes` to set its size.|defaults are the same as for the given cache type.| |`druid.cache.l2.*`|Prefix for L2 cache settings, see description for L1.|defaults are the same as for the given cache type.| -|`druid.cache.useL2`|A boolean indicating whether to query L2 cache, if it's a miss in L1. It makes sense to configure this to `false` on historical nodes, if L2 is a remote cache like `memcached`, and this cache also used on brokers, because in this case if a query reached historical it means that a broker didn't find corresponding results in the same remote cache, so a query to the remote cache from historical is guaranteed to be a miss.|`true`| +|`druid.cache.useL2`|A boolean indicating whether to query L2 cache, if it's a miss in L1. It makes sense to configure this to `false` on Historical nodes, if L2 is a remote cache like `memcached`, and this cache also used on brokers, because in this case if a query reached Historical it means that a broker didn't find corresponding results in the same remote cache, so a query to the remote cache from Historical is guaranteed to be a miss.|`true`| |`druid.cache.populateL2`|A boolean indicating whether to put results into L2 cache.|`true`| ## General Query Configuration -This section describes configurations that control behavior of Druid's query types, applicable to broker, historical, and middle manager nodes. +This section describes configurations that control behavior of Druid's query types, applicable to Broker, Historical, and MiddleManager nodes. ### TopN Query config @@ -1556,7 +1572,7 @@ This section describes configurations that control behavior of Druid's query typ ### GroupBy Query Config -This section describes the configurations for groupBy queries. You can set the runtime properties in the `runtime.properties` file on broker, historical, and MiddleManager nodes. You can set the query context parameters through the [query context](../querying/query-context.html). +This section describes the configurations for groupBy queries. You can set the runtime properties in the `runtime.properties` file on Broker, Historical, and MiddleManager nodes. You can set the query context parameters through the [query context](../querying/query-context.html). #### Configurations for groupBy v2 @@ -1616,7 +1632,7 @@ Supported query contexts: |`intermediateCombineDegree`|Overrides the value of `druid.query.groupBy.intermediateCombineDegree`|None| |`numParallelCombineThreads`|Overrides the value of `druid.query.groupBy.numParallelCombineThreads`|None| |`sortByDimsFirst`|Sort the results first by dimension values and then by timestamp.|false| -|`forceLimitPushDown`|When all fields in the orderby are part of the grouping key, the broker will push limit application down to the historical nodes. When the sorting order uses fields that are not in the grouping key, applying this optimization can result in approximate results with unknown accuracy, so this optimization is disabled by default in that case. Enabling this context flag turns on limit push down for limit/orderbys that contain non-grouping key columns.|false| +|`forceLimitPushDown`|When all fields in the orderby are part of the grouping key, the broker will push limit application down to the Historical nodes. When the sorting order uses fields that are not in the grouping key, applying this optimization can result in approximate results with unknown accuracy, so this optimization is disabled by default in that case. Enabling this context flag turns on limit push down for limit/orderbys that contain non-grouping key columns.|false| #### GroupBy v1 configurations diff --git a/docs/content/configuration/realtime.md b/docs/content/configuration/realtime.md index de636ec6e7c..396e1f7ee7e 100644 --- a/docs/content/configuration/realtime.md +++ b/docs/content/configuration/realtime.md @@ -61,7 +61,7 @@ The realtime node uses several of the global configs in [Configuration](../confi |Property|Description|Default| |--------|-----------|-------| |`druid.processing.buffer.sizeBytes`|This specifies a buffer size for the storage of intermediate results. The computation engine in both the Historical and Realtime nodes will use a scratch buffer of this size to do all of their intermediate computations off-heap. Larger values allow for more aggregations in a single pass over the data while smaller values can require more passes depending on the query that is being executed.|auto (max 1GB)| -|`druid.processing.formatString`|Realtime and historical nodes use this format string to name their processing threads.|processing-%s| +|`druid.processing.formatString`|Realtime and Historical nodes use this format string to name their processing threads.|processing-%s| |`druid.processing.numMergeBuffers`|The number of direct memory buffers available for merging query results. The buffers are sized by `druid.processing.buffer.sizeBytes`. This property is effectively a concurrency limit for queries that require merging buffers. If you are using any queries that require merge buffers (currently, just groupBy v2) then you should have at least two of these.|`max(2, druid.processing.numThreads / 4)`| |`druid.processing.numThreads`|The number of processing threads to have available for parallel processing of segments. Our rule of thumb is `num_cores - 1`, which means that even under heavy load there will still be one core available to do background tasks like talking with ZooKeeper and pulling down segments. If only one core is available, this property defaults to the value `1`.|Number of cores - 1 (or 1)| |`druid.processing.columnCache.sizeBytes`|Maximum size in bytes for the dimension value lookup cache. Any value greater than `0` enables the cache. It is currently disabled by default. Enabling the lookup cache can significantly improve the performance of aggregators operating on dimension values, such as the JavaScript aggregator, or cardinality aggregator, but can slow things down if the cache hit rate is low (i.e. dimensions with few repeating values). Enabling it may also require additional garbage collection tuning to avoid long GC pauses.|`0` (disabled)| diff --git a/docs/content/dependencies/cassandra-deep-storage.md b/docs/content/dependencies/cassandra-deep-storage.md index 681d298c98f..7eb4b1a1ffa 100644 --- a/docs/content/dependencies/cassandra-deep-storage.md +++ b/docs/content/dependencies/cassandra-deep-storage.md @@ -29,7 +29,7 @@ title: "Cassandra Deep Storage" Druid can use Cassandra as a deep storage mechanism. Segments and their metadata are stored in Cassandra in two tables: `index_storage` and `descriptor_storage`. Underneath the hood, the Cassandra integration leverages Astyanax. The index storage table is a [Chunked Object](https://github.com/Netflix/astyanax/wiki/Chunked-Object-Store) repository. It contains -compressed segments for distribution to historical nodes. Since segments can be large, the Chunked Object storage allows the integration to multi-thread +compressed segments for distribution to Historical nodes. Since segments can be large, the Chunked Object storage allows the integration to multi-thread the write to Cassandra, and spreads the data across all the nodes in a cluster. The descriptor storage table is a normal C* table that stores the segment metadatak. @@ -52,7 +52,7 @@ CREATE TABLE descriptor_storage(key varchar, First create the schema above. I use a new keyspace called `druid` for this purpose, which can be created using the [Cassandra CQL `CREATE KEYSPACE`](http://www.datastax.com/documentation/cql/3.1/cql/cql_reference/create_keyspace_r.html) command. -Then, add the following to your historical and realtime runtime properties files to enable a Cassandra backend. +Then, add the following to your Historical and realtime runtime properties files to enable a Cassandra backend. ```properties druid.extensions.loadList=["druid-cassandra-storage"] diff --git a/docs/content/dependencies/metadata-storage.md b/docs/content/dependencies/metadata-storage.md index f82ebfb5a02..bfc77cb7905 100644 --- a/docs/content/dependencies/metadata-storage.md +++ b/docs/content/dependencies/metadata-storage.md @@ -122,8 +122,7 @@ parameters across the cluster at runtime. ### Task-related Tables -There are also a number of tables created and used by the [Indexing -Service](../design/indexing-service.html) in the course of its work. +There are also a number of tables created and used by the [Overlord](../design/overlord.html) and [MiddleManager](../design/middlemanager.html) when managing tasks. ### Audit Table diff --git a/docs/content/dependencies/zookeeper.md b/docs/content/dependencies/zookeeper.md index b1ceeb569b9..d2b4cd04b73 100644 --- a/docs/content/dependencies/zookeeper.md +++ b/docs/content/dependencies/zookeeper.md @@ -29,8 +29,8 @@ Druid uses [ZooKeeper](http://zookeeper.apache.org/) (ZK) for management of curr 1. [Coordinator](../design/coordinator.html) leader election 2. Segment "publishing" protocol from [Historical](../design/historical.html) and [Realtime](../design/realtime.html) 3. Segment load/drop protocol between [Coordinator](../design/coordinator.html) and [Historical](../design/historical.html) -4. [Overlord](../design/indexing-service.html) leader election -5. [Indexing Service](../design/indexing-service.html) task management +4. [Overlord](../design/overlord.html) leader election +5. [Overlord](../design/overlord.html) and [MiddleManager](../design/middlemanager.html) task management ### Coordinator Leader Election @@ -74,4 +74,4 @@ When the [Coordinator](../design/coordinator.html) decides that a [Historical](. ${druid.zk.paths.loadQueuePath}/_host_of_historical_node/_segment_identifier ``` -This node will contain a payload that indicates to the historical node what it should do with the given segment. When the historical node is done with the work, it will delete the znode in order to signify to the Coordinator that it is complete. +This node will contain a payload that indicates to the Historical node what it should do with the given segment. When the Historical node is done with the work, it will delete the znode in order to signify to the Coordinator that it is complete. diff --git a/docs/content/design/auth.md b/docs/content/design/auth.md index c8b7a8c2a03..95f8be65ea9 100644 --- a/docs/content/design/auth.md +++ b/docs/content/design/auth.md @@ -80,7 +80,7 @@ druid.auth.authenticator.anonymous.authorizerName=myBasicAuthorizer ``` ## Escalator -The `druid.escalator.type` property determines what authentication scheme should be used for internal Druid cluster communications (such as when a broker node communicates with historical nodes for query processing). +The `druid.escalator.type` property determines what authentication scheme should be used for internal Druid cluster communications (such as when a Broker process communicates with Historical processes for query processing). The Escalator chosen for this property must use an authentication scheme that is supported by an Authenticator in `druid.auth.authenticationChain`. Authenticator extension implementors must also provide a corresponding Escalator implementation if they intend to use a particular authentication scheme for internal Druid communications. diff --git a/docs/content/design/broker.md b/docs/content/design/broker.md index ff90ffe5de6..1eb51cd16ce 100644 --- a/docs/content/design/broker.md +++ b/docs/content/design/broker.md @@ -47,9 +47,9 @@ org.apache.druid.cli.Main server broker Most druid queries contain an interval object that indicates a span of time for which data is requested. Likewise, Druid [Segments](../design/segments.html) are partitioned to contain data for some interval of time and segments are distributed across a cluster. Consider a simple datasource with 7 segments where each segment contains data for a given day of the week. Any query issued to the datasource for more than one day of data will hit more than one segment. These segments will likely be distributed across multiple nodes, and hence, the query will likely hit multiple nodes. -To determine which nodes to forward queries to, the Broker node first builds a view of the world from information in Zookeeper. Zookeeper maintains information about [Historical](../design/historical.html) and [Realtime](../design/realtime.html) nodes and the segments they are serving. For every datasource in Zookeeper, the Broker node builds a timeline of segments and the nodes that serve them. When queries are received for a specific datasource and interval, the Broker node performs a lookup into the timeline associated with the query datasource for the query interval and retrieves the nodes that contain data for the query. The Broker node then forwards down the query to the selected nodes. +To determine which nodes to forward queries to, the Broker node first builds a view of the world from information in Zookeeper. Zookeeper maintains information about [Historical](../design/historical.html) and [Realtime](../design/realtime.html) nodes and the segments they are serving. For every datasource in Zookeeper, the Broker node builds a timeline of segments and the nodes that serve them. When queries are received for a specific datasource and interval, the Broker node performs a lookup into the timeline associated with the query datasource for the query interval and retrieves the nodes that contain data for the query. The Broker process then forwards down the query to the selected nodes. ### Caching -Broker nodes employ a cache with a LRU cache invalidation strategy. The broker cache stores per-segment results. The cache can be local to each broker node or shared across multiple nodes using an external distributed cache such as [memcached](http://memcached.org/). Each time a broker node receives a query, it first maps the query to a set of segments. A subset of these segment results may already exist in the cache and the results can be directly pulled from the cache. For any segment results that do not exist in the cache, the broker node will forward the query to the -historical nodes. Once the historical nodes return their results, the broker will store those results in the cache. Real-time segments are never cached and hence requests for real-time data will always be forwarded to real-time nodes. Real-time data is perpetually changing and caching the results would be unreliable. +Broker nodes employ a cache with a LRU cache invalidation strategy. The Broker cache stores per-segment results. The cache can be local to each Broker process or shared across multiple processes using an external distributed cache such as [memcached](http://memcached.org/). Each time a broker node receives a query, it first maps the query to a set of segments. A subset of these segment results may already exist in the cache and the results can be directly pulled from the cache. For any segment results that do not exist in the cache, the broker node will forward the query to the +Historical nodes. Once the Historical processes return their results, the Broker will store those results in the cache. Real-time segments are never cached and hence requests for real-time data will always be forwarded to real-time nodes. Real-time data is perpetually changing and caching the results would be unreliable. diff --git a/docs/content/design/coordinator.md b/docs/content/design/coordinator.md index e8dea2908a9..a0fdb6b7913 100644 --- a/docs/content/design/coordinator.md +++ b/docs/content/design/coordinator.md @@ -34,11 +34,11 @@ For a list of API endpoints supported by the Coordinator, see [Coordinator API]( ### Overview -The Druid coordinator node is primarily responsible for segment management and distribution. More specifically, the Druid coordinator node communicates to historical nodes to load or drop segments based on configurations. The Druid coordinator is responsible for loading new segments, dropping outdated segments, managing segment replication, and balancing segment load. +The Druid Coordinator node is primarily responsible for segment management and distribution. More specifically, the Druid Coordinator node communicates to Historical nodes to load or drop segments based on configurations. The Druid Coordinator is responsible for loading new segments, dropping outdated segments, managing segment replication, and balancing segment load. -The Druid coordinator runs periodically and the time between each run is a configurable parameter. Each time the Druid coordinator runs, it assesses the current state of the cluster before deciding on the appropriate actions to take. Similar to the broker and historical nodes, the Druid coordinator maintains a connection to a Zookeeper cluster for current cluster information. The coordinator also maintains a connection to a database containing information about available segments and rules. Available segments are stored in a segment table and list all segments that should be loaded in the cluster. Rules are stored in a rule table and indicate how segments should be handled. +The Druid Coordinator runs periodically and the time between each run is a configurable parameter. Each time the Druid Coordinator runs, it assesses the current state of the cluster before deciding on the appropriate actions to take. Similar to the Broker and Historical processses, the Druid Coordinator maintains a connection to a Zookeeper cluster for current cluster information. The Coordinator also maintains a connection to a database containing information about available segments and rules. Available segments are stored in a segment table and list all segments that should be loaded in the cluster. Rules are stored in a rule table and indicate how segments should be handled. -Before any unassigned segments are serviced by historical nodes, the available historical nodes for each tier are first sorted in terms of capacity, with least capacity servers having the highest priority. Unassigned segments are always assigned to the nodes with least capacity to maintain a level of balance between nodes. The coordinator does not directly communicate with a historical node when assigning it a new segment; instead the coordinator creates some temporary information about the new segment under load queue path of the historical node. Once this request is seen, the historical node will load the segment and begin servicing it. +Before any unassigned segments are serviced by Historical nodes, the available Historical nodes for each tier are first sorted in terms of capacity, with least capacity servers having the highest priority. Unassigned segments are always assigned to the nodes with least capacity to maintain a level of balance between nodes. The Coordinator does not directly communicate with a historical node when assigning it a new segment; instead the Coordinator creates some temporary information about the new segment under load queue path of the historical node. Once this request is seen, the historical node will load the segment and begin servicing it. ### Running @@ -52,23 +52,23 @@ Segments can be automatically loaded and dropped from the cluster based on a set ### Cleaning Up Segments -Each run, the Druid coordinator compares the list of available database segments in the database with the current segments in the cluster. Segments that are not in the database but are still being served in the cluster are flagged and appended to a removal list. Segments that are overshadowed (their versions are too old and their data has been replaced by newer segments) are also dropped. -Note that if all segments in database are deleted(or marked unused), then coordinator will not drop anything from the historicals. This is done to prevent a race condition in which the coordinator would drop all segments if it started running cleanup before it finished polling the database for available segments for the first time and believed that there were no segments. +Each run, the Druid Coordinator compares the list of available database segments in the database with the current segments in the cluster. Segments that are not in the database but are still being served in the cluster are flagged and appended to a removal list. Segments that are overshadowed (their versions are too old and their data has been replaced by newer segments) are also dropped. +Note that if all segments in database are deleted(or marked unused), then Coordinator will not drop anything from the Historicals. This is done to prevent a race condition in which the Coordinator would drop all segments if it started running cleanup before it finished polling the database for available segments for the first time and believed that there were no segments. ### Segment Availability -If a historical node restarts or becomes unavailable for any reason, the Druid coordinator will notice a node has gone missing and treat all segments served by that node as being dropped. Given a sufficient period of time, the segments may be reassigned to other historical nodes in the cluster. However, each segment that is dropped is not immediately forgotten. Instead, there is a transitional data structure that stores all dropped segments with an associated lifetime. The lifetime represents a period of time in which the coordinator will not reassign a dropped segment. Hence, if a historical node becomes unavailable and available again within a short period of time, the historical node will start up and serve segments from its cache without any those segments being reassigned across the cluster. +If a Historical node restarts or becomes unavailable for any reason, the Druid Coordinator will notice a node has gone missing and treat all segments served by that node as being dropped. Given a sufficient period of time, the segments may be reassigned to other Historical nodes in the cluster. However, each segment that is dropped is not immediately forgotten. Instead, there is a transitional data structure that stores all dropped segments with an associated lifetime. The lifetime represents a period of time in which the Coordinator will not reassign a dropped segment. Hence, if a historical node becomes unavailable and available again within a short period of time, the historical node will start up and serve segments from its cache without any those segments being reassigned across the cluster. ### Balancing Segment Load -To ensure an even distribution of segments across historical nodes in the cluster, the coordinator node will find the total size of all segments being served by every historical node each time the coordinator runs. For every historical node tier in the cluster, the coordinator node will determine the historical node with the highest utilization and the historical node with the lowest utilization. The percent difference in utilization between the two nodes is computed, and if the result exceeds a certain threshold, a number of segments will be moved from the highest utilized node to the lowest utilized node. There is a configurable limit on the number of segments that can be moved from one node to another each time the coordinator runs. Segments to be moved are selected at random and only moved if the resulting utilization calculation indicates the percentage difference between the highest and lowest servers has decreased. +To ensure an even distribution of segments across Historical nodes in the cluster, the Coordinator node will find the total size of all segments being served by every Historical node each time the Coordinator runs. For every Historical node tier in the cluster, the Coordinator node will determine the Historical node with the highest utilization and the Historical node with the lowest utilization. The percent difference in utilization between the two nodes is computed, and if the result exceeds a certain threshold, a number of segments will be moved from the highest utilized node to the lowest utilized node. There is a configurable limit on the number of segments that can be moved from one node to another each time the Coordinator runs. Segments to be moved are selected at random and only moved if the resulting utilization calculation indicates the percentage difference between the highest and lowest servers has decreased. ### Compacting Segments -Each run, the Druid coordinator compacts small segments abutting each other. This is useful when you have a lot of small +Each run, the Druid Coordinator compacts small segments abutting each other. This is useful when you have a lot of small segments which may degrade the query performance as well as increasing the disk space usage. -The coordinator first finds the segments to compact together based on the [segment search policy](#segment-search-policy). +The Coordinator first finds the segments to compact together based on the [segment search policy](#segment-search-policy). Once some segments are found, it launches a [compaction task](../ingestion/tasks.html#compaction-task) to compact those segments. The maximum number of running compaction tasks is `min(sum of worker capacity * slotRatio, maxSlots)`. Note that even though `min(sum of worker capacity * slotRatio, maxSlots)` = 0, at least one compaction task is always submitted @@ -80,7 +80,7 @@ Compaction tasks might fail due to the following reasons. - If the input segments of a compaction task are removed or overshadowed before it starts, that compaction task fails immediately. - If a task of a higher priority acquires a lock for an interval overlapping with the interval of a compaction task, the compaction task fails. -Once a compaction task fails, the coordinator simply finds the segments for the interval of the failed task again, and launches a new compaction task in the next run. +Once a compaction task fails, the Coordinator simply finds the segments for the interval of the failed task again, and launches a new compaction task in the next run. ### Segment Search Policy @@ -113,28 +113,28 @@ If it finds such segments, it simply skips them. ### The Coordinator Console -The Druid coordinator exposes a web GUI for displaying cluster information and rule configuration. After the coordinator starts, the console can be accessed at: +The Druid Coordinator exposes a web GUI for displaying cluster information and rule configuration. After the Coordinator starts, the console can be accessed at: ``` http://: ``` - There exists a full cluster view (which shows only the realtime and historical nodes), as well as views for individual historical nodes, datasources and segments themselves. Segment information can be displayed in raw JSON form or as part of a sortable and filterable table. + There exists a full cluster view (which shows only the realtime and Historical nodes), as well as views for individual Historical nodes, datasources and segments themselves. Segment information can be displayed in raw JSON form or as part of a sortable and filterable table. -The coordinator console also exposes an interface to creating and editing rules. All valid datasources configured in the segment database, along with a default datasource, are available for configuration. Rules of different types can be added, deleted or edited. +The Coordinator console also exposes an interface to creating and editing rules. All valid datasources configured in the segment database, along with a default datasource, are available for configuration. Rules of different types can be added, deleted or edited. ### FAQ -1. **Do clients ever contact the coordinator node?** +1. **Do clients ever contact the Coordinator node?** - The coordinator is not involved in a query. + The Coordinator is not involved in a query. - historical nodes never directly contact the coordinator node. The Druid coordinator tells the historical nodes to load/drop data via Zookeeper, but the historical nodes are completely unaware of the coordinator. + Historical nodes never directly contact the Coordinator node. The Druid Coordinator tells the Historical nodes to load/drop data via Zookeeper, but the Historical nodes are completely unaware of the Coordinator. - Brokers also never contact the coordinator. Brokers base their understanding of the data topology on metadata exposed by the historical nodes via ZK and are completely unaware of the coordinator. + Brokers also never contact the Coordinator. Brokers base their understanding of the data topology on metadata exposed by the Historical nodes via ZK and are completely unaware of the Coordinator. -2. **Does it matter if the coordinator node starts up before or after other processes?** +2. **Does it matter if the Coordinator node starts up before or after other processes?** - No. If the Druid coordinator is not started up, no new segments will be loaded in the cluster and outdated segments will not be dropped. However, the coordinator node can be started up at any time, and after a configurable delay, will start running coordinator tasks. + No. If the Druid Coordinator is not started up, no new segments will be loaded in the cluster and outdated segments will not be dropped. However, the Coordinator node can be started up at any time, and after a configurable delay, will start running Coordinator tasks. - This also means that if you have a working cluster and all of your coordinators die, the cluster will continue to function, it just won’t experience any changes to its data topology. + This also means that if you have a working cluster and all of your Coordinators die, the cluster will continue to function, it just won’t experience any changes to its data topology. diff --git a/docs/content/design/historical.md b/docs/content/design/historical.md index fa7e79f7d4a..5d4efb407b5 100644 --- a/docs/content/design/historical.md +++ b/docs/content/design/historical.md @@ -40,20 +40,20 @@ org.apache.druid.cli.Main server historical ### Loading and Serving Segments -Each historical node maintains a constant connection to Zookeeper and watches a configurable set of Zookeeper paths for new segment information. Historical nodes do not communicate directly with each other or with the coordinator nodes but instead rely on Zookeeper for coordination. +Each Historical node maintains a constant connection to Zookeeper and watches a configurable set of Zookeeper paths for new segment information. Historical nodes do not communicate directly with each other or with the Coordinator nodes but instead rely on Zookeeper for coordination. -The [Coordinator](../design/coordinator.html) node is responsible for assigning new segments to historical nodes. Assignment is done by creating an ephemeral Zookeeper entry under a load queue path associated with a historical node. For more information on how the coordinator assigns segments to historical nodes, please see [Coordinator](../design/coordinator.html). +The [Coordinator](../design/coordinator.html) node is responsible for assigning new segments to Historical nodes. Assignment is done by creating an ephemeral Zookeeper entry under a load queue path associated with a Historical node. For more information on how the Coordinator assigns segments to Historical nodes, please see [Coordinator](../design/coordinator.html). -When a historical node notices a new load queue entry in its load queue path, it will first check a local disk directory (cache) for the information about segment. If no information about the segment exists in the cache, the historical node will download metadata about the new segment to serve from Zookeeper. This metadata includes specifications about where the segment is located in deep storage and about how to decompress and process the segment. For more information about segment metadata and Druid segments in general, please see [Segments](../design/segments.html). Once a historical node completes processing a segment, the segment is announced in Zookeeper under a served segments path associated with the node. At this point, the segment is available for querying. +When a Historical node notices a new load queue entry in its load queue path, it will first check a local disk directory (cache) for the information about segment. If no information about the segment exists in the cache, the Historical node will download metadata about the new segment to serve from Zookeeper. This metadata includes specifications about where the segment is located in deep storage and about how to decompress and process the segment. For more information about segment metadata and Druid segments in general, please see [Segments](../design/segments.html). Once a Historical node completes processing a segment, the segment is announced in Zookeeper under a served segments path associated with the node. At this point, the segment is available for querying. ### Loading and Serving Segments From Cache -Recall that when a historical node notices a new segment entry in its load queue path, the historical node first checks a configurable cache directory on its local disk to see if the segment had been previously downloaded. If a local cache entry already exists, the historical node will directly read the segment binary files from disk and load the segment. +Recall that when a Historical node notices a new segment entry in its load queue path, the Historical node first checks a configurable cache directory on its local disk to see if the segment had been previously downloaded. If a local cache entry already exists, the Historical node will directly read the segment binary files from disk and load the segment. -The segment cache is also leveraged when a historical node is first started. On startup, a historical node will search through its cache directory and immediately load and serve all segments that are found. This feature allows historical nodes to be queried as soon they come online. +The segment cache is also leveraged when a Historical node is first started. On startup, a Historical node will search through its cache directory and immediately load and serve all segments that are found. This feature allows Historical nodes to be queried as soon they come online. ### Querying Segments -Please see [Querying](../querying/querying.html) for more information on querying historical nodes. +Please see [Querying](../querying/querying.html) for more information on querying Historical nodes. -A historical can be configured to log and report metrics for every query it services. +A Historical can be configured to log and report metrics for every query it services. diff --git a/docs/content/design/index.md b/docs/content/design/index.md index 29c1aee2d2a..60a5a2bc5ed 100644 --- a/docs/content/design/index.md +++ b/docs/content/design/index.md @@ -93,48 +93,62 @@ Druid has a multi-process, distributed architecture that is designed to be cloud Druid process type can be configured and scaled independently, giving you maximum flexibility over your cluster. This design also provides enhanced fault tolerance: an outage of one component will not immediately affect other components. -Druid's process types are: +## Processes and Servers -* [**Historical**](../design/historical.html) processes are the workhorses that handle storage and querying on "historical" data -(including any streaming data that has been in the system long enough to be committed). Historical processes -download segments from deep storage and respond to queries about these segments. They don't accept writes. -* [**MiddleManager**](../design/middlemanager.html) processes handle ingestion of new data into the cluster. They are responsible -for reading from external data sources and publishing new Druid segments. -* [**Broker**](../design/broker.html) processes receive queries from external clients and forward those queries to Historicals and -MiddleManagers. When Brokers receive results from those subqueries, they merge those results and return them to the -caller. End users typically query Brokers rather than querying Historicals or MiddleManagers directly. -* [**Coordinator**](../design/coordinator.html) processes watch over the Historical processes. They are responsible for assigning -segments to specific servers, and for ensuring segments are well-balanced across Historicals. -* [**Overlord**](../design/overlord.html) processes watch over the MiddleManager processes and are the controllers of data ingestion -into Druid. They are responsible for assigning ingestion tasks to MiddleManagers and for coordinating segment -publishing. -* [**Router**](../development/router.html) processes are _optional_ processes that provide a unified API gateway in front of Druid Brokers, -Overlords, and Coordinators. They are optional since you can also simply contact the Druid Brokers, Overlords, and -Coordinators directly. +Druid has several process types, briefly described below: -Druid processes can be deployed individually (one per physical server, virtual server, or container) or can be colocated -on shared servers. One common colocation plan is a three-type plan: +* [**Coordinator**](../design/coordinator.html) processes manage data availability on the cluster. +* [**Overlord**](../design/overlord.html) processes control the assignment of data ingestion workloads. +* [**Broker**](../design/broker.html) processes handle queries from external clients. +* [**Router**](../development/router.html) processes are optional processes that can route requests to Brokers, Coordinators, and Overlords. +* [**Historical**](../design/historical.html) processes store queryable data. +* [**MiddleManager**](../design/middlemanager.html) processes are responsible for ingesting data. -1. "Data" servers run Historical and MiddleManager processes. -2. "Query" servers run Broker and (optionally) Router processes. -3. "Master" servers run Coordinator and Overlord processes. They may run ZooKeeper as well. +Druid processes can be deployed any way you like, but for ease of deployment we suggest organizing them into three server types: Master, Query, and Data. -In addition to these process types, Druid also has three external dependencies. These are intended to be able to +* **Master**: Runs Coordinator and Overlord processes, manages data availability and ingestion. +* **Query**: Runs Broker and optional Router processes, handles queries from external clients. +* **Data**: Runs Historical and MiddleManager processes, executes ingestion workloads and stores all queryable data. + +For more details on process and server organization, please see [Druid Processses and Servers](../design/processes.html). + +### External dependencies + +In addition to its built-in process types, Druid also has three external dependencies. These are intended to be able to leverage existing infrastructure, where present. -* [**Deep storage**](#deep-storage), shared file storage accessible by every Druid server. This is typically going to +#### Deep storage +Shared file storage accessible by every Druid server. This is typically going to be a distributed object store like S3 or HDFS, or a network mounted filesystem. Druid uses this to store any data that has been ingested into the system. -* [**Metadata store**](#metadata-storage), shared metadata storage. This is typically going to be a traditional RDBMS -like PostgreSQL or MySQL. -* [**ZooKeeper**](#zookeeper) is used for internal service discovery, coordination, and leader election. + +Druid uses deep storage only as a backup of your data and as a way to transfer data in the background between +Druid processes. To respond to queries, Historical processes do not read from deep storage, but instead read pre-fetched +segments from their local disks before any queries are served. This means that Druid never needs to access deep storage +during a query, helping it offer the best query latencies possible. It also means that you must have enough disk space +both in deep storage and across your Historical processes for the data you plan to load. + +For more details, please see [Deep storage dependency](../dependencies/deep-storage.html). + +#### Metadata storage +The metadata storage holds various shared system metadata such as segment availability information and task information. This is typically going to be a traditional RDBMS +like PostgreSQL or MySQL. + +For more details, please see [Metadata storage dependency](../dependencies/metadata-storage.html) + +#### Zookeeper +Used for internal service discovery, coordination, and leader election. + +For more details, please see [Zookeeper dependency](../dependencies/zookeeper.html). The idea behind this architecture is to make a Druid cluster simple to operate in production at scale. For example, the separation of deep storage and the metadata store from the rest of the cluster means that Druid processes are radically fault tolerant: even if every single Druid server fails, you can still relaunch your cluster from data stored in deep storage and the metadata store. -The following diagram shows how queries and data flow through this architecture: +### Architecture diagram + +The following diagram shows how queries and data flow through this architecture, using the suggested Master/Query/Data server organization: @@ -186,29 +200,4 @@ So Druid uses three different techniques to maximize query performance: - Pruning which segments are accessed for each query. - Within each segment, using indexes to identify which rows must be accessed. -- Within each segment, only reading the specific rows and columns that are relevant to a particular query. - - -# External Dependencies - -## Deep storage - -Druid uses deep storage only as a backup of your data and as a way to transfer data in the background between -Druid processes. To respond to queries, Historical processes do not read from deep storage, but instead read pre-fetched -segments from their local disks before any queries are served. This means that Druid never needs to access deep storage -during a query, helping it offer the best query latencies possible. It also means that you must have enough disk space -both in deep storage and across your Historical processes for the data you plan to load. - -For more details, please see [Deep storage dependency](../dependencies/deep-storage.html). - -## Metadata storage - -The metadata storage holds various system metadata such as segment availability information and task information. - -For more details, please see [Metadata storage dependency](../dependencies/metadata-storage.html) - -## Zookeeper - -Druid uses [ZooKeeper](http://zookeeper.apache.org/) (ZK) for management of current cluster state. - -For more details, please see [Zookeeper dependency](../dependencies/zookeeper.html). +- Within each segment, only reading the specific rows and columns that are relevant to a particular query. \ No newline at end of file diff --git a/docs/content/design/indexing-service.md b/docs/content/design/indexing-service.md index f11ab5edefd..0d583b7fb6c 100644 --- a/docs/content/design/indexing-service.md +++ b/docs/content/design/indexing-service.md @@ -28,8 +28,8 @@ The indexing service is a highly-available, distributed service that runs indexi Indexing tasks [tasks](../ingestion/tasks.html) create (and sometimes destroy) Druid [segments](../design/segments.html). The indexing service has a master/slave like architecture. -The indexing service is composed of three main components: a [Peon](../design/peons.html) component that can run a single task, a [Middle Manager](../design/middlemanager.html) component that manages peons, and an [Overlord](../design/overlord.html) component that manages task distribution to middle managers. -Overlords and middle managers may run on the same node or across multiple nodes while middle managers and peons always run on the same node. +The indexing service is composed of three main components: a [Peon](../design/peons.html) component that can run a single task, a [Middle Manager](../design/middlemanager.html) component that manages Peons, and an [Overlord](../design/overlord.html) component that manages task distribution to MiddleManagers. +Overlords and MiddleManagers may run on the same node or across multiple nodes while MiddleManagers and Peons always run on the same node. Tasks are managed using API endpoints on the Overlord service. Please see [Overlord Task API](../operations/api-reference.html#overlord-tasks) for more information. diff --git a/docs/content/design/middlemanager.md b/docs/content/design/middlemanager.md index 8d0f47fce48..fa5b1914219 100644 --- a/docs/content/design/middlemanager.md +++ b/docs/content/design/middlemanager.md @@ -34,8 +34,8 @@ For a list of API endpoints supported by the MiddleManager, please see the [API ### Overview -The middle manager node is a worker node that executes submitted tasks. Middle Managers forward tasks to peons that run in separate JVMs. -The reason we have separate JVMs for tasks is for resource and log isolation. Each [Peon](../design/peons.html) is capable of running only one task at a time, however, a middle manager may have multiple peons. +The MiddleManager node is a worker node that executes submitted tasks. Middle Managers forward tasks to Peons that run in separate JVMs. +The reason we have separate JVMs for tasks is for resource and log isolation. Each [Peon](../design/peons.html) is capable of running only one task at a time, however, a MiddleManager may have multiple Peons. ### Running diff --git a/docs/content/design/overlord.md b/docs/content/design/overlord.md index d96c98dcb62..fb5cf5662b8 100644 --- a/docs/content/design/overlord.md +++ b/docs/content/design/overlord.md @@ -34,14 +34,14 @@ For a list of API endpoints supported by the Overlord, please see the [API refer ### Overview -The overlord node is responsible for accepting tasks, coordinating task distribution, creating locks around tasks, and returning statuses to callers. Overlord can be configured to run in one of two modes - local or remote (local being default). -In local mode overlord is also responsible for creating peons for executing tasks. When running the overlord in local mode, all middle manager and peon configurations must be provided as well. -Local mode is typically used for simple workflows. In remote mode, the overlord and middle manager are run in separate processes and you can run each on a different server. +The Overlord node is responsible for accepting tasks, coordinating task distribution, creating locks around tasks, and returning statuses to callers. Overlord can be configured to run in one of two modes - local or remote (local being default). +In local mode Overlord is also responsible for creating Peons for executing tasks. When running the Overlord in local mode, all MiddleManager and Peon configurations must be provided as well. +Local mode is typically used for simple workflows. In remote mode, the Overlord and MiddleManager are run in separate processes and you can run each on a different server. This mode is recommended if you intend to use the indexing service as the single endpoint for all Druid indexing. ### Overlord Console -The overlord console can be used to view pending tasks, running tasks, available workers, and recent worker creation and termination. The console can be accessed at: +The Overlord console can be used to view pending tasks, running tasks, available workers, and recent worker creation and termination. The console can be accessed at: ``` http://:/console.html @@ -49,7 +49,7 @@ http://:/console.html ### Blacklisted Workers -If the workers fail tasks above a threshold, the overlord will blacklist these workers. No more than 20% of the nodes can be blacklisted. Blacklisted nodes will be periodically whitelisted. +If the workers fail tasks above a threshold, the Overlord will blacklist these workers. No more than 20% of the nodes can be blacklisted. Blacklisted nodes will be periodically whitelisted. The following vairables can be used to set the threshold and blacklist timeouts. @@ -62,6 +62,6 @@ druid.indexer.runner.maxPercentageBlacklistWorkers ### Autoscaling -The Autoscaling mechanisms currently in place are tightly coupled with our deployment infrastructure but the framework should be in place for other implementations. We are highly open to new implementations or extensions of the existing mechanisms. In our own deployments, middle manager nodes are Amazon AWS EC2 nodes and they are provisioned to register themselves in a [galaxy](https://github.com/ning/galaxy) environment. +The Autoscaling mechanisms currently in place are tightly coupled with our deployment infrastructure but the framework should be in place for other implementations. We are highly open to new implementations or extensions of the existing mechanisms. In our own deployments, MiddleManager nodes are Amazon AWS EC2 nodes and they are provisioned to register themselves in a [galaxy](https://github.com/ning/galaxy) environment. -If autoscaling is enabled, new middle managers may be added when a task has been in pending state for too long. Middle managers may be terminated if they have not run any tasks for a period of time. +If autoscaling is enabled, new MiddleManagers may be added when a task has been in pending state for too long. Middle managers may be terminated if they have not run any tasks for a period of time. diff --git a/docs/content/design/peons.md b/docs/content/design/peons.md index 7008a1095ba..5af3fd8ace0 100644 --- a/docs/content/design/peons.md +++ b/docs/content/design/peons.md @@ -37,7 +37,7 @@ Peons should rarely (if ever for testing purposes) be run on their own. ### Running -The peon should very rarely ever be run independent of the middle manager unless for development purposes. +The Peon should very rarely ever be run independent of the MiddleManager unless for development purposes. ``` org.apache.druid.cli.Main internal peon diff --git a/docs/content/design/processes.md b/docs/content/design/processes.md new file mode 100644 index 00000000000..87730ecfce6 --- /dev/null +++ b/docs/content/design/processes.md @@ -0,0 +1,129 @@ +--- +layout: doc_page +title: "Druid Processes and Servers" +--- + + + +# Druid Processes and Servers + +## Process Types + +Druid has several process types: + +* [Coordinator](../design/coordinator.html) +* [Overlord](../design/overlord.html) +* [Broker](../design/broker.html) +* [Historical](../design/historical.html) +* [MiddleManager](../design/middlemanager.html) and [Peons](../design/peons.html) +* [Router (Optional)](../development/router.html) + +## Server Types + +Druid processes can be deployed any way you like, but for ease of deployment we suggest organizing them into three server types: + +* **Master** +* **Query** +* **Data** + + + +This section describes the Druid processes and the suggested Master/Query/Data server organization, as shown in the architecture diagram above. + +### Master server + +A Master server manages data ingestion and availability: it is responsible for starting new ingestion jobs and coordinating availability of data on the "Data servers" described below. + +Within a Master server, functionality is split between two processes, the Coordinator and Overlord. + +#### Coordinator process + +[**Coordinator**](../design/coordinator.html) processes watch over the Historical processes on the Data servers. They are responsible for assigning segments to specific servers, and for ensuring segments are well-balanced across Historicals. + +#### Overlord process + +[**Overlord**](../design/overlord.html) processes watch over the MiddleManager processes on the Data servers and are the controllers of data ingestion into Druid. They are responsible for assigning ingestion tasks to MiddleManagers and for coordinating segment publishing. + +### Query server + +A Query server provides the endpoints that users and client applications interact with, routing queries to Data servers or other Query servers (and optionally proxied Master server requests as well). + +Within a Query server, functionality is split between two processes, the Broker and Router. + +#### Broker process + +[**Broker**](../design/broker.html) processes receive queries from external clients and forward those queries to Data servers. When Brokers receive results from those subqueries, they merge those results and return them to the +caller. End users typically query Brokers rather than querying Historicals or MiddleManagers processes on Data servers directly. + +#### Router process (optional) + +[**Router**](../development/router.html) processes are _optional_ processes that provide a unified API gateway in front of Druid Brokers, +Overlords, and Coordinators. They are optional since you can also simply contact the Druid Brokers, Overlords, and +Coordinators directly. + +### Data server + +A Data server executes ingestion jobs and stores queryable data. + +Within a Data server, functionality is split between two processes, the Historical and MiddleManager. + +### Historical process + +[**Historical**](../design/historical.html) processes are the workhorses that handle storage and querying on "historical" data +(including any streaming data that has been in the system long enough to be committed). Historical processes +download segments from deep storage and respond to queries about these segments. They don't accept writes. + +### Middle Manager process + +[**MiddleManager**](../design/middlemanager.html) processes handle ingestion of new data into the cluster. They are responsible +for reading from external data sources and publishing new Druid segments. + +#### Peon processes + +[**Peon**](../design/peons.html) processes are task execution engines spawned by MiddleManagers. Each Peon runs a separate JVM and is responsible for executing a single task. Peons always run on the same host as the MiddleManager that spawned them. + +## Pros and cons of colocation + +Druid processes can be colocated based on the Master/Data/Query server organization as +described above. This organization generally results in better utilization of +hardware resources for most clusters. + +For very large scale clusters, however, it can be desirable to split the Druid processes +such that they run on individual servers to avoid resource contention. + +This section describes guidelines and configuration parameters related to process colocation. + +### Coordinators and Overlords + +The workload on the Coordinator process tends to increase with the number of segments in the cluster. The Overlord's workload also increases based on the number of segments in the cluster, but to a lesser degree than the Coordinator. + +In clusters with very high segment counts, it can make sense to separate the Coordinator and Overlord processes to provide more resources for the Coordinator's segment balancing workload. + +#### Unified Process + +The Coordinator and Overlord processes can be run as a single combined process by setting the `druid.coordinator.asOverlord.enabled` property. + +Please see [Coordinator Configuration: Operation](../configuration/index.html#coordinator-operation) for details. + +### Historicals and MiddleManagers + +With higher levels of ingestion or query load, it can make sense to deploy the Historical and MiddleManager processes on separate nodes to to avoid CPU and memory contention. + +The Historical also benefits from having free memory for memory mapped segments, which can be another reason to deploy the Historical and MiddleManager processes separately. \ No newline at end of file diff --git a/docs/content/development/extensions-contrib/materialized-view.md b/docs/content/development/extensions-contrib/materialized-view.md index 01198a379d2..65fa689503e 100644 --- a/docs/content/development/extensions-contrib/materialized-view.md +++ b/docs/content/development/extensions-contrib/materialized-view.md @@ -24,7 +24,7 @@ title: "Materialized View" # Materialized View -To use this feature, make sure to only load materialized-view-selection on broker and load materialized-view-maintenance on overlord. In addtion, this feature currently requires a hadoop cluster. +To use this feature, make sure to only load `materialized-view-selection` on Broker and load `materialized-view-maintenance` on Overlord. In addtion, this feature currently requires a Hadoop cluster. This feature enables Druid to greatly improve the query performance, especially when the query dataSource has a very large number of dimensions but the query only required several dimensions. This feature includes two parts. One is `materialized-view-maintenance`, and the other is `materialized-view-selection`. @@ -76,7 +76,7 @@ A sample derivativeDataSource supervisor spec is shown below: |tuningConfig |TuningConfig must be HadoopTuningConfig. See [Hadoop tuning config](../../ingestion/hadoop.html#tuningconfig).|yes| |dataSource |The name of this derived dataSource. |no(default=baseDataSource-hashCode of supervisor)| |hadoopDependencyCoordinates |A JSON array of Hadoop dependency coordinates that Druid will use, this property will override the default Hadoop coordinates. Once specified, Druid will look for those Hadoop dependencies from the location specified by druid.extensions.hadoopDependenciesDir |no| -|classpathPrefix |Classpath that will be pre-appended for the peon process. |no| +|classpathPrefix |Classpath that will be pre-appended for the Peon process. |no| |context |See below. |no| **Context** diff --git a/docs/content/development/extensions-core/druid-basic-security.md b/docs/content/development/extensions-core/druid-basic-security.md index c8129298ded..890b2738d3a 100644 --- a/docs/content/development/extensions-core/druid-basic-security.md +++ b/docs/content/development/extensions-core/druid-basic-security.md @@ -43,10 +43,10 @@ These configuration properties should be added to the common runtime properties ### Properties |Property|Description|Default|required| |--------|-----------|-------|--------| -|`druid.auth.basic.common.pollingPeriod`|Defines in milliseconds how often nodes should poll the coordinator for the current authenticator/authorizer database state.|60000|No| +|`druid.auth.basic.common.pollingPeriod`|Defines in milliseconds how often nodes should poll the Coordinator for the current authenticator/authorizer database state.|60000|No| |`druid.auth.basic.common.maxRandomDelay`|Defines in milliseconds the amount of random delay to add to the pollingPeriod, to spread polling requests across time.|6000|No| -|`druid.auth.basic.common.maxSyncRetries`|Determines how many times a service will retry if the authentication/authorization database state sync with the coordinator fails.|10|No| -|`druid.auth.basic.common.cacheDirectory`|If defined, snapshots of the basic Authenticator and Authorizer database caches will be stored on disk in this directory. If this property is defined, when a service is starting, it will attempt to initialize its caches from these on-disk snapshots, if the service is unable to initialize its state by communicating with the coordinator.|null|No| +|`druid.auth.basic.common.maxSyncRetries`|Determines how many times a service will retry if the authentication/authorization database state sync with the Coordinator fails.|10|No| +|`druid.auth.basic.common.cacheDirectory`|If defined, snapshots of the basic Authenticator and Authorizer database caches will be stored on disk in this directory. If this property is defined, when a service is starting, it will attempt to initialize its caches from these on-disk snapshots, if the service is unable to initialize its state by communicating with the Coordinator.|null|No| ### Creating an Authenticator @@ -75,7 +75,7 @@ The configuration examples in the rest of this document will use "MyBasicAuthent |--------|-----------|-------|--------| |`druid.auth.authenticator.MyBasicAuthenticator.initialAdminPassword`|Initial [Password Provider](../../operations/password-provider.html) for the automatically created default admin user. If no password is specified, the default admin user will not be created. If the default admin user already exists, setting this property will not affect its password.|null|No| |`druid.auth.authenticator.MyBasicAuthenticator.initialInternalClientPassword`|Initial [Password Provider](../../operations/password-provider.html) for the default internal system user, used for internal node communication. If no password is specified, the default internal system user will not be created. If the default internal system user already exists, setting this property will not affect its password.|null|No| -|`druid.auth.authenticator.MyBasicAuthenticator.enableCacheNotifications`|If true, the coordinator will notify Druid nodes whenever a configuration change to this Authenticator occurs, allowing them to immediately update their state without waiting for polling.|true|No| +|`druid.auth.authenticator.MyBasicAuthenticator.enableCacheNotifications`|If true, the Coordinator will notify Druid nodes whenever a configuration change to this Authenticator occurs, allowing them to immediately update their state without waiting for polling.|true|No| |`druid.auth.authenticator.MyBasicAuthenticator.cacheNotificationTimeout`|The timeout in milliseconds for the cache notifications.|5000|No| |`druid.auth.authenticator.MyBasicAuthenticator.credentialIterations`|Number of iterations to use for password hashing.|10000|No| |`druid.auth.authenticator.MyBasicAuthenticator.authorizerName`|Authorizer that requests should be directed to|N/A|Yes| @@ -116,7 +116,7 @@ druid.auth.authorizer.. #### Properties |Property|Description|Default|required| |--------|-----------|-------|--------| -|`druid.auth.authorizer.MyBasicAuthorizer.enableCacheNotifications`|If true, the coordinator will notify Druid nodes whenever a configuration change to this Authorizer occurs, allowing them to immediately update their state without waiting for polling.|true|No| +|`druid.auth.authorizer.MyBasicAuthorizer.enableCacheNotifications`|If true, the Coordinator will notify Druid nodes whenever a configuration change to this Authorizer occurs, allowing them to immediately update their state without waiting for polling.|true|No| |`druid.auth.authorizer.MyBasicAuthorizer.cacheNotificationTimeout`|The timeout in milliseconds for the cache notifications.|5000|No| ## Usage @@ -302,7 +302,7 @@ There is only one possible resource name for the "STATE" config resource type, " |`/druid-internal/v1/segments/`|historical| |`/druid-internal/v1/segments/`|peon| |`/druid-internal/v1/segments/`|realtime| -|`/status`|all nodes| +|`/status`|all process types| ### HTTP methods @@ -312,10 +312,10 @@ GET requires READ permission, while POST and DELETE require WRITE permission. ## Configuration Propagation -To prevent excessive load on the coordinator, the Authenticator and Authorizer user/role database state is cached on each Druid node. +To prevent excessive load on the Coordinator, the Authenticator and Authorizer user/role database state is cached on each Druid node. -Each node will periodically poll the coordinator for the latest database state, controlled by the `druid.auth.basic.common.pollingPeriod` and `druid.auth.basic.common.maxRandomDelay` properties. +Each node will periodically poll the Coordinator for the latest database state, controlled by the `druid.auth.basic.common.pollingPeriod` and `druid.auth.basic.common.maxRandomDelay` properties. -When a configuration update occurs, the coordinator can optionally notify each node with the updated database state. This behavior is controlled by the `enableCacheNotifications` and `cacheNotificationTimeout` properties on Authenticators and Authorizers. +When a configuration update occurs, the Coordinator can optionally notify each node with the updated database state. This behavior is controlled by the `enableCacheNotifications` and `cacheNotificationTimeout` properties on Authenticators and Authorizers. Note that because of the caching, changes made to the user/role database may not be immediately reflected at each Druid node. diff --git a/docs/content/development/extensions-core/druid-kerberos.md b/docs/content/development/extensions-core/druid-kerberos.md index decd384c412..c97004a00eb 100644 --- a/docs/content/development/extensions-core/druid-kerberos.md +++ b/docs/content/development/extensions-core/druid-kerberos.md @@ -98,14 +98,14 @@ Druid internal nodes communicate with each other using an escalated http Client. curl --negotiate -u:anyUser -b ~/cookies.txt -c ~/cookies.txt -X POST -H'Content-Type: application/json' ``` - e.g to send a query from file `query.json` to druid broker use this command - + e.g to send a query from file `query.json` to the Druid Broker use this command - ``` curl --negotiate -u:anyUser -b ~/cookies.txt -c ~/cookies.txt -X POST -H'Content-Type: application/json' http://broker-host:port/druid/v2/?pretty -d @query.json ``` Note: Above command will authenticate the user first time using SPNego negotiate mechanism and store the authentication cookie in file. For subsequent requests the cookie will be used for authentication. -## Accessing coordinator or overlord console from web browser +## Accessing Coordinator or Overlord console from web browser To access Coordinator/Overlord console from browser you will need to configure your browser for SPNego authentication as follows - 1. Safari - No configurations required. diff --git a/docs/content/development/extensions-core/kafka-ingestion.md b/docs/content/development/extensions-core/kafka-ingestion.md index af391bbae73..4753585f945 100644 --- a/docs/content/development/extensions-core/kafka-ingestion.md +++ b/docs/content/development/extensions-core/kafka-ingestion.md @@ -44,8 +44,8 @@ version 0.10.x or better before using this service. Refer "}| +|`druid.router.defaultBrokerServiceName`|Any string.|The default Broker to connect to in case service discovery fails.|druid/broker| +|`druid.router.tierToBrokerMap`|An ordered JSON map of tiers to Broker names. The priority of Brokers is based on the ordering.|Queries for a certain tier of data are routed to their appropriate Broker.|{"_default_tier": ""}| |`druid.router.defaultRule`|Any string.|The default rule for all datasources.|"_default"| |`druid.router.pollPeriod`|Any ISO8601 duration.|How often to poll for new rules.|PT1M| |`druid.router.strategies`|An ordered JSON array of objects.|All custom strategies to use for routing.|[{"type":"timeBoundary"},{"type":"priority"}]| -|`druid.router.avatica.balancer.type`|String representing an AvaticaConnectionBalancer name|Class to use for balancing Avatica queries across brokers|rendezvousHash| -|`druid.router.http.maxRequestBufferSize`|Maximum size of the buffer used to write requests when forwarding them to the broker. This should be set to atleast the maxHeaderSize allowed on the broker|8 * 1024| +|`druid.router.avatica.balancer.type`|String representing an AvaticaConnectionBalancer name|Class to use for balancing Avatica queries across Brokers|rendezvousHash| +|`druid.router.http.maxRequestBufferSize`|Maximum size of the buffer used to write requests when forwarding them to the Broker. This should be set to atleast the maxHeaderSize allowed on the Broker|8 * 1024| Router Strategies ----------------- -The router has a configurable list of strategies for how it selects which brokers to route queries to. The order of the strategies matter because as soon as a strategy condition is matched, a broker is selected. +The Router has a configurable list of strategies for how it selects which Brokers to route queries to. The order of the strategies matter because as soon as a strategy condition is matched, a Broker is selected. ### timeBoundary @@ -108,7 +108,7 @@ The router has a configurable list of strategies for how it selects which broker } ``` -Including this strategy means all timeBoundary queries are always routed to the highest priority broker. +Including this strategy means all timeBoundary queries are always routed to the highest priority Broker. ### priority @@ -120,13 +120,13 @@ Including this strategy means all timeBoundary queries are always routed to the } ``` -Queries with a priority set to less than minPriority are routed to the lowest priority broker. Queries with priority set to greater than maxPriority are routed to the highest priority broker. By default, minPriority is 0 and maxPriority is 1. Using these default values, if a query with priority 0 (the default query priority is 0) is sent, the query skips the priority selection logic. +Queries with a priority set to less than minPriority are routed to the lowest priority Broker. Queries with priority set to greater than maxPriority are routed to the highest priority Broker. By default, minPriority is 0 and maxPriority is 1. Using these default values, if a query with priority 0 (the default query priority is 0) is sent, the query skips the priority selection logic. ### JavaScript Allows defining arbitrary routing rules using a JavaScript function. The function is passed the configuration and the query to be executed, and returns the tier it should be routed to, or null for the default tier. -*Example*: a function that sends queries containing more than three aggregators to the lowest priority broker. +*Example*: a function that sends queries containing more than three aggregators to the lowest priority Broker. ```json { @@ -143,15 +143,15 @@ JavaScript-based functionality is disabled by default. Please refer to the Druid Avatica Query Balancing -------------- -All Avatica JDBC requests with a given connection ID must be routed to the same broker, since Druid brokers do not share connection state with each other. +All Avatica JDBC requests with a given connection ID must be routed to the same Broker, since Druid Brokers do not share connection state with each other. -To accomplish this, Druid provides two built-in balancers that use rendezvous hashing and consistent hashing of a request's connection ID respectively to assign requests to brokers. +To accomplish this, Druid provides two built-in balancers that use rendezvous hashing and consistent hashing of a request's connection ID respectively to assign requests to Brokers. -Note that when multiple routers are used, all routers should have identical balancer configuration to ensure that they make the same routing decisions. +Note that when multiple Routers are used, all Routers should have identical balancer configuration to ensure that they make the same routing decisions. ### Rendezvous Hash Balancer -This balancer uses [Rendezvous Hashing](https://en.wikipedia.org/wiki/Rendezvous_hashing) on an Avatica request's connection ID to assign the request to a broker. +This balancer uses [Rendezvous Hashing](https://en.wikipedia.org/wiki/Rendezvous_hashing) on an Avatica request's connection ID to assign the request to a Broker. To use this balancer, specify the following property: @@ -163,7 +163,7 @@ If no `druid.router.avatica.balancer` property is set, the Router will also defa ### Consistent Hash Balancer -This balancer uses [Consistent Hashing](https://en.wikipedia.org/wiki/Consistent_hashing) on an Avatica request's connection ID to assign the request to a broker. +This balancer uses [Consistent Hashing](https://en.wikipedia.org/wiki/Consistent_hashing) on an Avatica request's connection ID to assign the request to a Broker. To use this balancer, specify the following property: @@ -171,12 +171,12 @@ To use this balancer, specify the following property: druid.router.avatica.balancer.type=consistentHash ``` -This is a non-default implementation that is provided for experimentation purposes. The consistent hasher has longer setup times on initialization and when the set of brokers changes, but has a faster broker assignment time than the rendezous hasher when tested with 5 brokers. Benchmarks for both implementations have been provided in `ConsistentHasherBenchmark` and `RendezvousHasherBenchmark`. The consistent hasher also requires locking, while the rendezvous hasher does not. +This is a non-default implementation that is provided for experimentation purposes. The consistent hasher has longer setup times on initialization and when the set of Brokers changes, but has a faster Broker assignment time than the rendezous hasher when tested with 5 Brokers. Benchmarks for both implementations have been provided in `ConsistentHasherBenchmark` and `RendezvousHasherBenchmark`. The consistent hasher also requires locking, while the rendezvous hasher does not. HTTP Endpoints -------------- -The router node exposes several HTTP endpoints for interactions. +The Router node exposes several HTTP endpoints for interactions. ### GET @@ -203,14 +203,14 @@ Returns the metrics of the datasource. Router as Management Proxy -------------------------- -The router can be configured to forward requests to the active coordinator or overlord node. This may be useful for +The Router can be configured to forward requests to the active Coordinator or Overlord node. This may be useful for setting up a highly available cluster in situations where the HTTP redirect mechanism of the inactive -> active -coordinator/overlord does not function correctly (servers are behind a load balancer, the hostname used in the redirect +Coordinator/Overlord does not function correctly (servers are behind a load balancer, the hostname used in the redirect is only resolvable internally, etc.). ### Enabling the Management Proxy -To enable this functionality, set the following in the router's runtime.properties: +To enable this functionality, set the following in the Router's runtime.properties: ``` druid.router.managementProxy.enabled=true @@ -219,13 +219,13 @@ druid.router.managementProxy.enabled=true ### Routing The management proxy supports implicit and explicit routes. Implicit routes are those where the destination can be -determined from the original request path based on Druid API path conventions. For the coordinator the convention is -`/druid/coordinator/*` and for the overlord the convention is `/druid/indexer/*`. These are convenient because they mean -that using the management proxy does not require modifying the API request other than issuing the request to the router -instead of the coordinator or overlord. Most Druid API requests can be routed implicitly. +determined from the original request path based on Druid API path conventions. For the Coordinator the convention is +`/druid/coordinator/*` and for the Overlord the convention is `/druid/indexer/*`. These are convenient because they mean +that using the management proxy does not require modifying the API request other than issuing the request to the Router +instead of the Coordinator or Overlord. Most Druid API requests can be routed implicitly. -Explicit routes are those where the request to the router contains a path prefix indicating which node the request -should be routed to. For the coordinator this prefix is `/proxy/coordinator` and for the overlord it is `/proxy/overlord`. +Explicit routes are those where the request to the Router contains a path prefix indicating which node the request +should be routed to. For the Coordinator this prefix is `/proxy/coordinator` and for the Overlord it is `/proxy/overlord`. This is required for API calls with an ambiguous destination. For example, the `/status` API is present on all Druid nodes, so explicit routing needs to be used to indicate the proxy destination. diff --git a/docs/content/ingestion/faq.md b/docs/content/ingestion/faq.md index 13a0a95f3d9..e9584e8b133 100644 --- a/docs/content/ingestion/faq.md +++ b/docs/content/ingestion/faq.md @@ -56,19 +56,19 @@ Other common reasons that hand-off fails are as follows: 1) Druid is unable to write to the metadata storage. Make sure your configurations are correct. -2) Historical nodes are out of capacity and cannot download any more segments. You'll see exceptions in the coordinator logs if this occurs and the coordinator console will show the historicals are near capacity. +2) Historical nodes are out of capacity and cannot download any more segments. You'll see exceptions in the Coordinator logs if this occurs and the Coordinator console will show the Historicals are near capacity. -3) Segments are corrupt and cannot be downloaded. You'll see exceptions in your historical nodes if this occurs. +3) Segments are corrupt and cannot be downloaded. You'll see exceptions in your Historical nodes if this occurs. -4) Deep storage is improperly configured. Make sure that your segment actually exists in deep storage and that the coordinator logs have no errors. +4) Deep storage is improperly configured. Make sure that your segment actually exists in deep storage and that the Coordinator logs have no errors. ## How do I get HDFS to work? Make sure to include the `druid-hdfs-storage` and all the hadoop configuration, dependencies (that can be obtained by running command `hadoop classpath` on a machine where hadoop has been setup) in the classpath. And, provide necessary HDFS settings as described in [Deep Storage](../dependencies/deep-storage.html) . -## I don't see my Druid segments on my historical nodes +## I don't see my Druid segments on my Historical nodes -You can check the coordinator console located at `:`. Make sure that your segments have actually loaded on [historical nodes](../design/historical.html). If your segments are not present, check the coordinator logs for messages about capacity of replication errors. One reason that segments are not downloaded is because historical nodes have maxSizes that are too small, making them incapable of downloading more data. You can change that with (for example): +You can check the Coordinator console located at `:`. Make sure that your segments have actually loaded on [Historical nodes](../design/historical.html). If your segments are not present, check the Coordinator logs for messages about capacity of replication errors. One reason that segments are not downloaded is because Historical nodes have maxSizes that are too small, making them incapable of downloading more data. You can change that with (for example): ``` -Ddruid.segmentCache.locations=[{"path":"/tmp/druid/storageLocation","maxSize":"500000000000"}] diff --git a/docs/content/ingestion/hadoop.md b/docs/content/ingestion/hadoop.md index 7ae0eaeeaec..4f8174c40a9 100644 --- a/docs/content/ingestion/hadoop.md +++ b/docs/content/ingestion/hadoop.md @@ -25,7 +25,7 @@ title: "Hadoop-based Batch Ingestion" # Hadoop-based Batch Ingestion Hadoop-based batch ingestion in Druid is supported via a Hadoop-ingestion task. These tasks can be posted to a running -instance of a Druid [overlord](../design/overlord.html). +instance of a Druid [Overlord](../design/overlord.html). ## Command Line Hadoop Indexer @@ -105,7 +105,7 @@ A sample task is shown below: |type|The task type, this should always be "index_hadoop".|yes| |spec|A Hadoop Index Spec. See [Ingestion](../ingestion/ingestion-spec.html)|yes| |hadoopDependencyCoordinates|A JSON array of Hadoop dependency coordinates that Druid will use, this property will override the default Hadoop coordinates. Once specified, Druid will look for those Hadoop dependencies from the location specified by `druid.extensions.hadoopDependenciesDir`|no| -|classpathPrefix|Classpath that will be pre-appended for the peon process.|no| +|classpathPrefix|Classpath that will be pre-appended for the Peon process.|no| also note that, druid automatically computes the classpath for hadoop job containers that run in hadoop cluster. But, in case of conflicts between hadoop and druid's dependencies, you can manually specify the classpath by setting `druid.extensions.hadoopContainerDruidClasspath` property. See the extensions config in [base druid configuration](../configuration/index.html#extensions). diff --git a/docs/content/ingestion/index.md b/docs/content/ingestion/index.md index fa473d9c2b9..db1edfa0172 100644 --- a/docs/content/ingestion/index.md +++ b/docs/content/ingestion/index.md @@ -291,9 +291,9 @@ For compaction documentation, please see [tasks](../ingestion/tasks.html). Druid supports retention rules, which are used to define intervals of time where data should be preserved, and intervals where data should be discarded. -Druid also supports separating historical nodes into tiers, and the retention rules can be configured to assign data for specific intervals to specific tiers. +Druid also supports separating Historical nodes into tiers, and the retention rules can be configured to assign data for specific intervals to specific tiers. -These features are useful for performance/cost management; a common use case is separating historical nodes into a "hot" tier and a "cold" tier. +These features are useful for performance/cost management; a common use case is separating Historical nodes into a "hot" tier and a "cold" tier. For more information, please see [Load rules](../operations/rule-configuration.html). diff --git a/docs/content/ingestion/locking-and-priority.md b/docs/content/ingestion/locking-and-priority.md index 523ed77b27d..09424661d6f 100644 --- a/docs/content/ingestion/locking-and-priority.md +++ b/docs/content/ingestion/locking-and-priority.md @@ -26,7 +26,7 @@ title: "Task Locking & Priority" ## Locking -Once an overlord node accepts a task, the task acquires locks for the data source and intervals specified in the task. +Once an Overlord node accepts a task, the task acquires locks for the data source and intervals specified in the task. There are two lock types, i.e., _shared lock_ and _exclusive lock_. @@ -75,5 +75,5 @@ The task context is used for various task configuration parameters. The followin
When a task acquires a lock, it sends a request via HTTP and awaits until it receives a response containing the lock acquisition result. -As a result, an HTTP timeout error can occur if `taskLockTimeout` is greater than `druid.server.http.maxIdleTime` of overlords. +As a result, an HTTP timeout error can occur if `taskLockTimeout` is greater than `druid.server.http.maxIdleTime` of Overlords.
diff --git a/docs/content/ingestion/native_tasks.md b/docs/content/ingestion/native_tasks.md index 1f2c4531914..e5b2e7d2871 100644 --- a/docs/content/ingestion/native_tasks.md +++ b/docs/content/ingestion/native_tasks.md @@ -25,8 +25,8 @@ title: "Native Index Tasks" # Native Index Tasks Druid currently has two types of native batch indexing tasks, `index_parallel` which runs tasks -in parallel on multiple middle manager nodes, and `index` which will run a single indexing task locally on a single -middle manager. +in parallel on multiple MiddleManager nodes, and `index` which will run a single indexing task locally on a single +MiddleManager. Parallel Index Task -------------------------------- @@ -34,7 +34,7 @@ Parallel Index Task The Parallel Index Task is a task for parallel batch indexing. This task only uses Druid's resource and doesn't depend on other external systems like Hadoop. This task currently works in a single phase without shuffling intermediate data. `index_parallel` task is a supervisor task which basically generates multiple worker tasks and submits -them to overlords. Each worker task reads input data and makes segments. Once they successfully generate segments for all +them to Overlords. Each worker task reads input data and makes segments. Once they successfully generate segments for all input, they report the generated segment list to the supervisor task. The supervisor task periodically checks the worker task statuses. If one of them fails, it retries the failed task until the retrying number reaches the configured limit. If all worker tasks succeed, then it collects the reported list of generated segments and publishes those segments at once. diff --git a/docs/content/ingestion/reports.md b/docs/content/ingestion/reports.md index de37cd136d2..869637499ce 100644 --- a/docs/content/ingestion/reports.md +++ b/docs/content/ingestion/reports.md @@ -90,7 +90,7 @@ The `errorMsg` field shows a message describing the error that caused a task to The non-parallel [Native Batch Task](../native_tasks.md), the Hadoop batch task, and the tasks created by the Kafka Indexing Service support retrieval of row stats while the task is running. -The live report can be accessed with a GET to the following URL on a peon running a task: +The live report can be accessed with a GET to the following URL on a Peon running a task: ``` http://:/druid/worker/v1/chat//rowStats @@ -143,7 +143,7 @@ http://:/druid/indexer/v1/supervisor/:/druid/worker/v1/chat//unparseableEvents diff --git a/docs/content/ingestion/stream-pull.md b/docs/content/ingestion/stream-pull.md index d3b74255c71..0ec8b7fd48b 100644 --- a/docs/content/ingestion/stream-pull.md +++ b/docs/content/ingestion/stream-pull.md @@ -336,7 +336,7 @@ data segments on each of the N nodes, which is not possible with standard Kafka For example, let's say your topic is split across Kafka partitions 1, 2, & 3 and you have 2 real-time nodes with linear shard specs 1 & 2. Both of the real-time nodes are in the same consumer group. Real-time node 1 may consume data from partitions 1 & 3, and real-time node 2 may consume data from partition 2. -Querying for your data through the broker will yield correct results. +Querying for your data through the Broker will yield correct results. The problem arises if you want to replicate your data by creating real-time nodes 3 & 4. These new real-time nodes also have linear shard specs 1 & 2, and they will consume data from Kafka using a different consumer group. In this case, diff --git a/docs/content/ingestion/stream-push.md b/docs/content/ingestion/stream-push.md index b31b3ab19fe..9552499f7ec 100644 --- a/docs/content/ingestion/stream-push.md +++ b/docs/content/ingestion/stream-push.md @@ -41,8 +41,8 @@ time). Older messages will not be processed in real-time. Historical data is bes ### Server Druid can use [Tranquility Server](https://github.com/druid-io/tranquility/blob/master/docs/server.md), which -lets you send data to Druid without developing a JVM app. You can run Tranquility server colocated with Druid middleManagers -and historical processes. +lets you send data to Druid without developing a JVM app. You can run Tranquility server colocated with Druid MiddleManagers +and Historical processes. Tranquility server is started by issuing: @@ -125,7 +125,7 @@ The windowPeriod is the slack time permitted for events. For example, a windowPe than ten minutes in the future, will be dropped. These are important configurations because they influence how long tasks will be alive for, and how -long data stays in the realtime system before being handed off to the historical nodes. For example, +long data stays in the realtime system before being handed off to the Historical nodes. For example, if your configuration has segmentGranularity "hour" and windowPeriod ten minutes, tasks will stay around listening for events for an hour and ten minutes. For this reason, to prevent excessive buildup of tasks, it is recommended that your windowPeriod be less than your segmentGranularity. diff --git a/docs/content/ingestion/tasks.md b/docs/content/ingestion/tasks.md index 4c55b0efc2a..41f7b52444b 100644 --- a/docs/content/ingestion/tasks.md +++ b/docs/content/ingestion/tasks.md @@ -24,7 +24,7 @@ title: "Tasks Overview" # Tasks Overview -Tasks are run on middle managers and always operate on a single data source. +Tasks are run on MiddleManagers and always operate on a single data source. Tasks are submitted using POST requests to the Overlord. Please see [Overlord Task API](../operations/api-reference.html#overlord-tasks) for API details. diff --git a/docs/content/ingestion/update-existing-data.md b/docs/content/ingestion/update-existing-data.md index 506e5b3926a..057572628cd 100644 --- a/docs/content/ingestion/update-existing-data.md +++ b/docs/content/ingestion/update-existing-data.md @@ -73,7 +73,7 @@ Here is what goes inside `ingestionSpec`: |-----|----|-----------|--------| |dataSource|String|Druid dataSource name from which you are loading the data.|yes| |intervals|List|A list of strings representing ISO-8601 Intervals.|yes| -|segments|List|List of segments from which to read data from, by default it is obtained automatically. You can obtain list of segments to put here by making a POST query to coordinator at url /druid/coordinator/v1/metadata/datasources/segments?full with list of intervals specified in the request paylod e.g. ["2012-01-01T00:00:00.000/2012-01-03T00:00:00.000", "2012-01-05T00:00:00.000/2012-01-07T00:00:00.000"]. You may want to provide this list manually in order to ensure that segments read are exactly same as they were at the time of task submission, task would fail if the list provided by the user does not match with state of database when the task actually runs.|no| +|segments|List|List of segments from which to read data from, by default it is obtained automatically. You can obtain list of segments to put here by making a POST query to Coordinator at url /druid/coordinator/v1/metadata/datasources/segments?full with list of intervals specified in the request paylod e.g. ["2012-01-01T00:00:00.000/2012-01-03T00:00:00.000", "2012-01-05T00:00:00.000/2012-01-07T00:00:00.000"]. You may want to provide this list manually in order to ensure that segments read are exactly same as they were at the time of task submission, task would fail if the list provided by the user does not match with state of database when the task actually runs.|no| |filter|JSON|See [Filters](../querying/filters.html)|no| |dimensions|Array of String|Name of dimension columns to load. By default, the list will be constructed from parseSpec. If parseSpec does not have an explicit list of dimensions then all the dimension columns present in stored data will be read.|no| |metrics|Array of String|Name of metric columns to load. By default, the list will be constructed from the "name" of all the configured aggregators.|no| @@ -148,7 +148,7 @@ For example: } ``` -It is STRONGLY RECOMMENDED to provide list of segments in `dataSource` inputSpec explicitly so that your delta ingestion task is idempotent. You can obtain that list of segments by making following call to the coordinator. +It is STRONGLY RECOMMENDED to provide list of segments in `dataSource` inputSpec explicitly so that your delta ingestion task is idempotent. You can obtain that list of segments by making following call to the Coordinator. POST `/druid/coordinator/v1/metadata/datasources/{dataSourceName}/segments?full` Request Body: [interval1, interval2,...] for example ["2012-01-01T00:00:00.000/2012-01-03T00:00:00.000", "2012-01-05T00:00:00.000/2012-01-07T00:00:00.000"] diff --git a/docs/content/operations/api-reference.md b/docs/content/operations/api-reference.md index 8389eb0ecdc..7a0b43421fc 100644 --- a/docs/content/operations/api-reference.md +++ b/docs/content/operations/api-reference.md @@ -28,12 +28,15 @@ This page documents all of the API endpoints for each Druid service type. ## Table of Contents * [Common](#common) - * [Coordinator](#coordinator) - * [Overlord](#overlord) - * [MiddleManager](#middlemanager) - * [Peon](#peon) - * [Broker](#broker) - * [Historical](#historical) + * [Master Server](#master-server) + * [Coordinator](#coordinator) + * [Overlord](#overlord) + * [Data Server](#data-server) + * [MiddleManager](#middlemanager) + * [Peon](#peon) + * [Historical](#historical) + * [Query Server](#query-server) + * [Broker](#broker) ## Common @@ -55,26 +58,30 @@ An endpoint that always returns a boolean "true" value with a 200 OK response, u Returns the current configuration properties of the node. -## Coordinator +## Master Server -### Leadership +This section documents the API endpoints for the processes that reside on Master servers (Coordinators and Overlords) in the suggested [three-server configuration](../design/processes.html#server-types). -#### GET +### Coordinator + +#### Leadership + +##### GET * `/druid/coordinator/v1/leader` -Returns the current leader coordinator of the cluster. +Returns the current leader Coordinator of the cluster. * `/druid/coordinator/v1/isLeader` -Returns true if the coordinator receiving the request is the current leader. +Returns true if the Coordinator receiving the request is the current leader. -### Segment Loading +#### Segment Loading -#### GET +##### GET Returns a JSON object with field "leader", either true or false, indicating if this server is the current leader -coordinator of the cluster. In addition, returns HTTP 200 if the server is the current leader and HTTP 404 if not. +Coordinator of the cluster. In addition, returns HTTP 200 if the server is the current leader and HTTP 404 if not. This is suitable for use as a load balancer status check if you only want the active leader to be considered in-service at the load balancer. @@ -92,19 +99,19 @@ Returns the number of segments left to load in each tier until segments that sho * `/druid/coordinator/v1/loadqueue` -Returns the ids of segments to load and drop for each historical node. +Returns the ids of segments to load and drop for each Historical node. * `/druid/coordinator/v1/loadqueue?simple` -Returns the number of segments to load and drop, as well as the total segment load and drop size in bytes for each historical node. +Returns the number of segments to load and drop, as well as the total segment load and drop size in bytes for each Historical node. * `/druid/coordinator/v1/loadqueue?full` -Returns the serialized JSON of segments to load and drop for each historical node. +Returns the serialized JSON of segments to load and drop for each Historical node. -### Metadata store information +#### Metadata store information -#### GET +##### GET * `/druid/coordinator/v1/metadata/datasources` @@ -134,7 +141,7 @@ Returns a list of all segments for a datasource with the full segment metadata a Returns full segment metadata for a specific segment as stored in the metadata store. -#### POST +##### POST * `/druid/coordinator/v1/metadata/datasources/{dataSourceName}/segments` @@ -144,10 +151,9 @@ Returns a list of all segments, overlapping with any of given intervals, for a Returns a list of all segments, overlapping with any of given intervals, for a datasource with the full segment metadata as stored in the metadata store. Request body is array of string intervals like [interval1, interval2,...] for example ["2012-01-01T00:00:00.000/2012-01-03T00:00:00.000", "2012-01-05T00:00:00.000/2012-01-07T00:00:00.000"] +#### Datasources -### Datasources - -#### GET +##### GET * `/druid/coordinator/v1/datasources` @@ -213,7 +219,7 @@ Returns full segment metadata for a specific segment in the cluster. Return the tiers that a datasource exists in. -#### POST +##### POST * `/druid/coordinator/v1/datasources/{dataSourceName}` @@ -223,7 +229,7 @@ Enables all segments of datasource which are not overshadowed by others. Enables a segment of a datasource. -#### DELETE
+##### DELETE * `/druid/coordinator/v1/datasources/{dataSourceName}` @@ -240,9 +246,9 @@ Note that {interval} parameters are delimited by a `_` instead of a `/` (e.g., 2 Disables a segment. -### Retention Rules +#### Retention Rules -#### GET +##### GET * `/druid/coordinator/v1/rules` @@ -259,7 +265,7 @@ Returns all rules for a specified datasource and includes default datasource. * `/druid/coordinator/v1/rules/history?interval=` - Returns audit history of rules for all datasources. default value of interval can be specified by setting `druid.audit.manager.auditHistoryMillis` (1 week if not configured) in coordinator runtime.properties + Returns audit history of rules for all datasources. default value of interval can be specified by setting `druid.audit.manager.auditHistoryMillis` (1 week if not configured) in Coordinator runtime.properties * `/druid/coordinator/v1/rules/history?count=` @@ -267,13 +273,13 @@ Returns all rules for a specified datasource and includes default datasource. * `/druid/coordinator/v1/rules/{dataSourceName}/history?interval=` - Returns audit history of rules for a specified datasource. default value of interval can be specified by setting `druid.audit.manager.auditHistoryMillis` (1 week if not configured) in coordinator runtime.properties + Returns audit history of rules for a specified datasource. default value of interval can be specified by setting `druid.audit.manager.auditHistoryMillis` (1 week if not configured) in Coordinator runtime.properties * `/druid/coordinator/v1/rules/{dataSourceName}/history?count=` Returns last entries of audit history of rules for a specified datasource. -#### POST +##### POST * `/druid/coordinator/v1/rules/{dataSourceName}` @@ -286,9 +292,9 @@ Optional Header Parameters for auditing the config change can also be specified. |`X-Druid-Author`| author making the config change|""| |`X-Druid-Comment`| comment describing the change being done|""| -### Intervals +#### Intervals -#### GET +##### GET Note that {interval} parameters are delimited by a `_` instead of a `/` (e.g., 2016-06-27_2016-06-28). @@ -308,9 +314,9 @@ Returns total size and count for each interval within given isointerval. Returns total size and count for each datasource for each interval within given isointerval. -### Compaction Configuration +#### Compaction Configuration -#### GET +##### GET * `/druid/coordinator/v1/config/compaction` @@ -320,7 +326,7 @@ Returns all compaction configs. Returns a compaction config of a dataSource. -#### POST +##### POST * `/druid/coordinator/v1/config/compaction/taskslots?ratio={someRatio}&max={someMaxSlots}` @@ -334,15 +340,15 @@ will be set for them. Creates or updates the compaction config for a dataSource. See [Compaction Configuration](../configuration/index.html#compaction-dynamic-configuration) for configuration details. -#### DELETE +##### DELETE * `/druid/coordinator/v1/config/compaction/{dataSource}` Removes the compaction config for a dataSource. -### Server Information +#### Server Information -#### GET +##### GET * `/druid/coordinator/v1/servers` @@ -360,15 +366,15 @@ Returns a list of server data objects in which each object has the following key - `priority` - `tier` -## Overlord +### Overlord -### Leadership +#### Leadership -#### GET +##### GET * `/druid/indexer/v1/leader` -Returns the current leader overlord of the cluster. If you have multiple overlords, just one is leading at any given time. The others are on standby. +Returns the current leader Overlord of the cluster. If you have multiple Overlords, just one is leading at any given time. The others are on standby. * `/druid/indexer/v1/isLeader` @@ -376,9 +382,9 @@ This returns a JSON object with field "leader", either true or false. In additio server is the current leader and HTTP 404 if not. This is suitable for use as a load balancer status check if you only want the active leader to be considered in-service at the load balancer. -### Tasks +#### Tasks -#### GET +##### GET * `/druid/indexer/v1/task/{taskId}/status` @@ -396,11 +402,11 @@ This API is deprecated and will be removed in future releases. Retrieve a [task completion report](../ingestion/reports.html) for a task. Only works for completed tasks. -#### POST +##### POST * `/druid/indexer/v1/task` -Endpoint for submitting tasks and supervisor specs to the overlord. Returns the taskId of the submitted task. +Endpoint for submitting tasks and supervisor specs to the Overlord. Returns the taskId of the submitted task. * `druid/indexer/v1/task/{taskId}/shutdown` @@ -410,9 +416,9 @@ Shuts down a task. Shuts down all tasks for a dataSource. -### Supervisors +#### Supervisors -#### GET +##### GET * `/druid/indexer/v1/supervisor` @@ -443,7 +449,7 @@ Returns an audit history of specs for all supervisors (current and past). Returns an audit history of specs for the supervisor with the provided ID. -#### POST +##### POST * `/druid/indexer/v1/supervisor` @@ -486,28 +492,53 @@ This API is deprecated and will be removed in future releases. Please use the equivalent 'terminate' instead. -## MiddleManager +## Data Server + +This section documents the API endpoints for the processes that reside on Data servers (MiddleManagers/Peons and Historicals) in the suggested [three-server configuration](../design/processes.html#server-types). + +### MiddleManager The MiddleManager does not have any API endpoints beyond the [common endpoints](#common). -## Peon +### Peon #### GET * `/druid/worker/v1/chat/{taskId}/rowStats` -Retrieve a live row stats report from a peon. See [task reports](../ingestion/reports.html) for more details. +Retrieve a live row stats report from a Peon. See [task reports](../ingestion/reports.html) for more details. * `/druid/worker/v1/chat/{taskId}/unparseableEvents` -Retrieve an unparseable events report from a peon. See [task reports](../ingestion/reports.html) for more details. +Retrieve an unparseable events report from a Peon. See [task reports](../ingestion/reports.html) for more details. + +### Historical + +#### Segment Loading + +##### GET + +* `/druid/historical/v1/loadstatus` + +Returns JSON of the form `{"cacheInitialized":}`, where value is either `true` or `false` indicating if all +segments in the local cache have been loaded. This can be used to know when a Historical node is ready +to be queried after a restart. + +* `/druid/historical/v1/readiness` + +Similar to `/druid/historical/v1/loadstatus`, but instead of returning JSON with a flag, responses 200 OK if segments +in the local cache have been loaded, and 503 SERVICE UNAVAILABLE, if they haven't. -## Broker +## Query Server -### Datasource Information +This section documents the API endpoints for the processes that reside on Query servers (Brokers) in the suggested [three-server configuration](../design/processes.html#server-types). -#### GET +### Broker + +#### Datasource Information + +##### GET * `/druid/v2/datasources` @@ -545,18 +576,17 @@ if you're using SQL. Returns segment information lists including server locations for the given datasource and intervals. If "numCandidates" is not specified, it will return all servers for each interval. -### Load Status +#### Load Status -#### GET +##### GET * `/druid/broker/v1/loadstatus` -Returns a flag indicating if the broker knows about all segments in Zookeeper. This can be used to know when a broker node is ready to be queried after a restart. +Returns a flag indicating if the Broker knows about all segments in Zookeeper. This can be used to know when a Broker process is ready to be queried after a restart. +#### Queries -### Queries - -#### POST +##### POST * `/druid/v2/` @@ -566,20 +596,3 @@ The endpoint for submitting queries. Accepts an option `?pretty` that pretty pri Returns segment information lists including server locations for the given query.. - -## Historical - -### Segment Loading - -#### GET - -* `/druid/historical/v1/loadstatus` - -Returns JSON of the form `{"cacheInitialized":}`, where value is either `true` or `false` indicating if all -segments in the local cache have been loaded. This can be used to know when a historical node is ready -to be queried after a restart. - -* `/druid/historical/v1/readiness` - -Similar to `/druid/historical/v1/loadstatus`, but instead of returning JSON with a flag, responses 200 OK if segments -in the local cache have been loaded, and 503 SERVICE UNAVAILABLE, if they haven't. diff --git a/docs/content/operations/metrics.md b/docs/content/operations/metrics.md index cb4056f913f..7bce8c00003 100644 --- a/docs/content/operations/metrics.md +++ b/docs/content/operations/metrics.md @@ -53,7 +53,7 @@ Available Metrics |`query/bytes`|number of bytes returned in query response.|Common: dataSource, type, interval, hasFilters, duration, context, remoteAddress, id. Aggregation Queries: numMetrics, numComplexMetrics. GroupBy: numDimensions. TopN: threshold, dimension.| | |`query/node/time`|Milliseconds taken to query individual historical/realtime nodes.|id, status, server.|< 1s| |`query/node/bytes`|number of bytes returned from querying individual historical/realtime nodes.|id, status, server.| | -|`query/node/ttfb`|Time to first byte. Milliseconds elapsed until broker starts receiving the response from individual historical/realtime nodes.|id, status, server.|< 1s| +|`query/node/ttfb`|Time to first byte. Milliseconds elapsed until Broker starts receiving the response from individual historical/realtime nodes.|id, status, server.|< 1s| |`query/node/backpressure`|Milliseconds that the channel to this node has spent suspended due to backpressure.|id, status, server.| | |`query/intervalChunk/time`|Only emitted if interval chunking is enabled. Milliseconds required to query an interval chunk. This metric is deprecated and will be removed in the future because interval chunking is deprecated. See [Query Context](../querying/query-context.html).|id, status, chunkInterval (if interval chunking is enabled).|< 1s| |`query/count`|number of total queries|This metric is only available if the QueryCountStatsMonitor module is included.|| @@ -69,7 +69,7 @@ Available Metrics |`query/segment/time`|Milliseconds taken to query individual segment. Includes time to page in the segment from disk.|id, status, segment.|several hundred milliseconds| |`query/wait/time`|Milliseconds spent waiting for a segment to be scanned.|id, segment.|< several hundred milliseconds| |`segment/scan/pending`|Number of segments in queue waiting to be scanned.||Close to 0| -|`query/segmentAndCache/time`|Milliseconds taken to query individual segment or hit the cache (if it is enabled on the historical node).|id, segment.|several hundred milliseconds| +|`query/segmentAndCache/time`|Milliseconds taken to query individual segment or hit the cache (if it is enabled on the Historical node).|id, segment.|several hundred milliseconds| |`query/cpu/time`|Microseconds of CPU time taken to complete a query|Common: dataSource, type, interval, hasFilters, duration, context, remoteAddress, id. Aggregation Queries: numMetrics, numComplexMetrics. GroupBy: numDimensions. TopN: threshold, dimension.|Varies| |`query/count`|number of total queries|This metric is only available if the QueryCountStatsMonitor module is included.|| |`query/success/count`|number of queries successfully processed|This metric is only available if the QueryCountStatsMonitor module is included.|| @@ -128,7 +128,7 @@ Memcached client metrics are reported as per the following. These metrics come d ## SQL Metrics -If SQL is enabled, broker will emit the following metrics for SQL. +If SQL is enabled, the Broker will emit the following metrics for SQL. |Metric|Description|Dimensions|Normal Value| |------|-----------|----------|------------| @@ -190,7 +190,7 @@ Note: If the JVM does not support CPU time measurement for the current thread, i ## Coordination -These metrics are for the Druid coordinator and are reset each time the coordinator runs the coordination logic. +These metrics are for the Druid Coordinator and are reset each time the Coordinator runs the coordination logic. |Metric|Description|Dimensions|Normal Value| |------|-----------|----------|------------| @@ -212,7 +212,7 @@ These metrics are for the Druid coordinator and are reset each time the coordina |`segment/unavailable/count`|Number of segments (not including replicas) left to load until segments that should be loaded in the cluster are available for queries.|datasource.|0| |`segment/underReplicated/count`|Number of segments (including replicas) left to load until segments that should be loaded in the cluster are available for queries.|tier, datasource.|0| -If `emitBalancingStats` is set to `true` in the coordinator [dynamic configuration](../configuration/index.html#dynamic-configuration), then [log entries](../configuration/logging.html) for class `org.apache.druid.server.coordinator.helper.DruidCoordinatorLogger` will have extra information on balancing decisions. +If `emitBalancingStats` is set to `true` in the Coordinator [dynamic configuration](../configuration/index.html#dynamic-configuration), then [log entries](../configuration/logging.html) for class `org.apache.druid.server.coordinator.helper.DruidCoordinatorLogger` will have extra information on balancing decisions. ## General Health diff --git a/docs/content/operations/other-hadoop.md b/docs/content/operations/other-hadoop.md index a7b49520bfd..74a86965f25 100644 --- a/docs/content/operations/other-hadoop.md +++ b/docs/content/operations/other-hadoop.md @@ -56,7 +56,7 @@ Generally, you should only set one of these parameters, not both. These properties can be set in either one of the following ways: - Using the task definition, e.g. add `"mapreduce.job.classloader": "true"` to the `jobProperties` of the `tuningConfig` of your indexing task (see the [Hadoop batch ingestion documentation](../ingestion/hadoop.html)). -- Using system properties, e.g. on the middleManager set `druid.indexer.runner.javaOpts=... -Dhadoop.mapreduce.job.classloader=true`. +- Using system properties, e.g. on the MiddleManager set `druid.indexer.runner.javaOpts=... -Dhadoop.mapreduce.job.classloader=true`. ### Overriding specific classes @@ -181,7 +181,7 @@ For more about building Druid, please see [Building Druid](../development/build. **Alternate workaround - 2** -Another workaround solution is to build a custom fat jar of Druid using [sbt](http://www.scala-sbt.org/), which manually excludes all the conflicting Jackson dependencies, and then put this fat jar in the classpath of the command that starts overlord indexing service. To do this, please follow the following steps. +Another workaround solution is to build a custom fat jar of Druid using [sbt](http://www.scala-sbt.org/), which manually excludes all the conflicting Jackson dependencies, and then put this fat jar in the classpath of the command that starts Overlord indexing service. To do this, please follow the following steps. (1) Download and install sbt. diff --git a/docs/content/operations/performance-faq.md b/docs/content/operations/performance-faq.md index 03b2c37b8b2..672ca9d3a40 100644 --- a/docs/content/operations/performance-faq.md +++ b/docs/content/operations/performance-faq.md @@ -32,16 +32,16 @@ Improper configuration is by far the largest problem we see people trying to dep The size of the JVM heap really depends on the type of Druid node you are running. Below are a few considerations. -[Broker nodes](../design/broker.html) uses the JVM heap mainly to merge results from historicals and real-times. Brokers also use off-heap memory and processing threads for groupBy queries. We recommend 20G-30G of heap here. +[Broker nodes](../design/broker.html) uses the JVM heap mainly to merge results from Historicals and real-times. Brokers also use off-heap memory and processing threads for groupBy queries. We recommend 20G-30G of heap here. -[Historical nodes](../design/historical.html) use off-heap memory to store intermediate results, and by default, all segments are memory mapped before they can be queried. Typically, the more memory is available on a historical node, the more segments can be served without the possibility of data being paged on to disk. On historicals, the JVM heap is used for [GroupBy queries](../querying/groupbyquery.html), some data structures used for intermediate computation, and general processing. One way to calculate how much space there is for segments is: memory_for_segments = total_memory - heap - direct_memory - jvm_overhead. Note that total_memory here refers to the memory available to the cgroup (if running on Linux), which for default cases is going to be all the system memory. +[Historical nodes](../design/historical.html) use off-heap memory to store intermediate results, and by default, all segments are memory mapped before they can be queried. Typically, the more memory is available on a Historical node, the more segments can be served without the possibility of data being paged on to disk. On Historicals, the JVM heap is used for [GroupBy queries](../querying/groupbyquery.html), some data structures used for intermediate computation, and general processing. One way to calculate how much space there is for segments is: memory_for_segments = total_memory - heap - direct_memory - jvm_overhead. Note that total_memory here refers to the memory available to the cgroup (if running on Linux), which for default cases is going to be all the system memory. We recommend 250mb * (processing.numThreads) for the heap. [Coordinator nodes](../design/coordinator.html) do not require off-heap memory and the heap is used for loading information about all segments to determine what segments need to be loaded, dropped, moved, or replicated. ## How much direct memory does Druid use? -Any Druid node that process queries (brokers, ingestion workers, and historical nodes) use two kinds of direct memory buffers with configurable size: processing buffers and merge buffers. +Any Druid process that handles queries (Brokers, Peons, and Historicals) uses two kinds of direct memory buffers with configurable size: processing buffers and merge buffers. Each processing thread is allocated one processing buffer. Additionally, there is a shared pool of merge buffers (only used for GroupBy V2 queries currently). diff --git a/docs/content/operations/recommendations.md b/docs/content/operations/recommendations.md index cfac2ae4c2f..bb0acd30974 100644 --- a/docs/content/operations/recommendations.md +++ b/docs/content/operations/recommendations.md @@ -70,7 +70,7 @@ We recommend using UTC timezone for all your events and across on your nodes, no # SSDs -SSDs are highly recommended for historical and real-time nodes if you are not running a cluster that is entirely in memory. SSDs can greatly mitigate the time required to page data in and out of memory. +SSDs are highly recommended for Historical and real-time nodes if you are not running a cluster that is entirely in memory. SSDs can greatly mitigate the time required to page data in and out of memory. # JBOD vs RAID Historical nodes store large number of segments on Disk and support specifying multiple paths for storing those. Typically, hosts have multiple disks configured with RAID which makes them look like a single disk to OS. RAID might have overheads specially if its not hardware controller based but software based. So, Historicals might get improved disk throughput with JBOD. diff --git a/docs/content/operations/rolling-updates.md b/docs/content/operations/rolling-updates.md index c154fc42928..9b908037ed3 100644 --- a/docs/content/operations/rolling-updates.md +++ b/docs/content/operations/rolling-updates.md @@ -82,7 +82,7 @@ to `/druid/worker/v1/enable`. If autoscaling is enabled on your Overlord, then Overlord nodes can launch new Middle Manager nodes en masse and then gracefully terminate old ones as their tasks finish. This process is configured by -setting `druid.indexer.runner.minWorkerVersion=#{VERSION}`. Each time you update your overlord node, +setting `druid.indexer.runner.minWorkerVersion=#{VERSION}`. Each time you update your Overlord node, the `VERSION` value should be increased, which will trigger a mass launch of new Middle Managers. The config `druid.indexer.autoscale.workerVersion=#{VERSION}` also needs to be set. @@ -94,7 +94,7 @@ Standalone real-time nodes can be updated one at a time in a rolling fashion. ## Broker Broker nodes can be updated one at a time in a rolling fashion. There needs to be some delay between -updating each node as brokers must load the entire state of the cluster before they return valid +updating each node as Brokers must load the entire state of the cluster before they return valid results. ## Coordinator diff --git a/docs/content/operations/rule-configuration.md b/docs/content/operations/rule-configuration.md index 9e67d5da993..4d67789986b 100644 --- a/docs/content/operations/rule-configuration.md +++ b/docs/content/operations/rule-configuration.md @@ -24,16 +24,16 @@ title: "Retaining or Automatically Dropping Data" # Retaining or Automatically Dropping Data -Coordinator nodes use rules to determine what data should be loaded to or dropped from the cluster. Rules are used for data retention and query execution, and are set on the coordinator console (http://coordinator_ip:port). +Coordinator nodes use rules to determine what data should be loaded to or dropped from the cluster. Rules are used for data retention and query execution, and are set on the Coordinator console (http://coordinator_ip:port). There are three types of rules, i.e., load rules, drop rules, and broadcast rules. Load rules indicate how segments should be assigned to different historical node tiers and how many replicas of a segment should exist in each tier. Drop rules indicate when segments should be dropped entirely from the cluster. Finally, broadcast rules indicate how segments of different data sources should be co-located in historical nodes. -The coordinator loads a set of rules from the metadata storage. Rules may be specific to a certain datasource and/or a default set of rules can be configured. Rules are read in order and hence the ordering of rules is important. The coordinator will cycle through all available segments and match each segment with the first rule that applies. Each segment may only match a single rule. +The Coordinator loads a set of rules from the metadata storage. Rules may be specific to a certain datasource and/or a default set of rules can be configured. Rules are read in order and hence the ordering of rules is important. The Coordinator will cycle through all available segments and match each segment with the first rule that applies. Each segment may only match a single rule. -Note: It is recommended that the coordinator console is used to configure rules. However, the coordinator node does have HTTP endpoints to programmatically configure rules. +Note: It is recommended that the Coordinator console is used to configure rules. However, the Coordinator node does have HTTP endpoints to programmatically configure rules. -When a rule is updated, the change may not be reflected until the next time the coordinator runs. This will be fixed in the near future. +When a rule is updated, the change may not be reflected until the next time the Coordinator runs. This will be fixed in the near future. Load Rules ---------- @@ -173,7 +173,7 @@ The interval of a segment will be compared against the specified period. The per Broadcast Rules --------------- -Broadcast rules indicate how segments of different data sources should be co-located in historical nodes. +Broadcast rules indicate how segments of different data sources should be co-located in Historical nodes. Once a broadcast rule is configured for a data source, all segments of the data source are broadcasted to the servers holding _any segments_ of the co-located data sources. ### Forever Broadcast Rule @@ -234,9 +234,9 @@ If you want to always co-locate the segments of some data sources together, it i # Permanently Deleting Data Druid can fully drop data from the cluster, wipe the metadata store entry, and remove the data from deep storage for any segments that are -marked as unused (segments dropped from the cluster via rules are always marked as unused). You can submit a [kill task](../ingestion/tasks.html) to the [indexing service](../design/indexing-service.html) to do this. +marked as unused (segments dropped from the cluster via rules are always marked as unused). You can submit a [kill task](../ingestion/tasks.html) to the [Overlord](../design/overlord.html) to do this. # Reloading Dropped Data Data that has been dropped from a Druid cluster cannot be reloaded using only rules. To reload dropped data in Druid, you must first set your retention period (i.e. changing the retention period from 1 month to 2 months), and -then enable the datasource in the Druid coordinator console, or through the Druid coordinator endpoints. +then enable the datasource in the Druid Coordinator console, or through the Druid Coordinator endpoints. diff --git a/docs/content/operations/segment-optimization.md b/docs/content/operations/segment-optimization.md index 04d099e65ee..2fea3ff3d9c 100644 --- a/docs/content/operations/segment-optimization.md +++ b/docs/content/operations/segment-optimization.md @@ -28,7 +28,7 @@ In Druid, it's important to optimize the segment size because 1. Druid stores data in segments. If you're using the [best-effort roll-up](../design/index.html#roll-up-modes) mode, increasing the segment size might introduce further aggregation which reduces the dataSource size. - 2. When a query is submitted, that query is distributed to all historicals and realtimes + 2. When a query is submitted, that query is distributed to all Historicals and realtimes which hold the input segments of the query. Each node has a processing threads pool and use one thread per segment to process it. If the segment size is too large, data might not be well distributed over the whole cluster, thereby decreasing the degree of parallelism. If the segment size is too small, @@ -39,8 +39,8 @@ It would be best if you can optimize the segment size at ingestion time, but som especially for the streaming ingestion because the amount of data ingested might vary over time. In this case, you can roughly set the segment size at ingestion time and optimize it later. You have two options: - - Turning on the [automatic compaction of coordinators](../design/coordinator.html#compacting-segments). - The coordinator periodically submits [compaction tasks](../ingestion/tasks.html#compaction-task) to re-index small segments. + - Turning on the [automatic compaction of Coordinators](../design/coordinator.html#compacting-segments). + The Coordinator periodically submits [compaction tasks](../ingestion/tasks.html#compaction-task) to re-index small segments. - Running periodic Hadoop batch ingestion jobs and using a `dataSource` inputSpec to read from the segments generated by the Kafka indexing tasks. This might be helpful if you want to compact a lot of segments in parallel. Details on how to do this can be found under ['Updating Existing Data'](../ingestion/update-existing-data.html). diff --git a/docs/content/querying/caching.md b/docs/content/querying/caching.md index cb84e1f1448..c83d0678857 100644 --- a/docs/content/querying/caching.md +++ b/docs/content/querying/caching.md @@ -34,7 +34,7 @@ can be enabled at either the Historical and Broker level (it is not recommended ## Query caching on Brokers -Enabling caching on the broker can yield faster results than if query caches were enabled on Historicals for small clusters. This is +Enabling caching on the Broker can yield faster results than if query caches were enabled on Historicals for small clusters. This is the recommended setup for smaller production clusters (< 20 servers). Take note that when caching is enabled on the Broker, results from Historicals are returned on a per segment basis, and Historicals will not be able to do any local result merging. Result level caching is enabled only on the Broker side. diff --git a/docs/content/querying/datasource.md b/docs/content/querying/datasource.md index d9811e8a7f4..552e1cc5060 100644 --- a/docs/content/querying/datasource.md +++ b/docs/content/querying/datasource.md @@ -48,7 +48,7 @@ This data source unions two or more table data sources. ``` Note that the data sources being unioned should have the same schema. -Union Queries should be always sent to the broker/router node and are *NOT* supported directly by the historical nodes. +Union Queries should be always sent to a Broker/Router process and are *NOT* supported directly by the Historical processes. ### Query Data Source diff --git a/docs/content/querying/dimensionspecs.md b/docs/content/querying/dimensionspecs.md index c6c636dc5f8..ce57f468b4d 100644 --- a/docs/content/querying/dimensionspecs.md +++ b/docs/content/querying/dimensionspecs.md @@ -123,7 +123,7 @@ It is illegal to set `retainMissingValue = true` and also specify a `replaceMiss A property `optimize` can be supplied to allow optimization of lookup based extraction filter (by default `optimize = true`). -The second kind where it is not possible to pass at query time due to their size, will be based on an external lookup table or resource that is already registered via configuration file or/and coordinator. +The second kind where it is not possible to pass at query time due to their size, will be based on an external lookup table or resource that is already registered via configuration file or/and Coordinator. ```json { @@ -375,7 +375,7 @@ A property of `injective` can override the lookup's own sense of whether or not configuration. A property `optimize` can be supplied to allow optimization of lookup based extraction filter (by default `optimize = true`). -The optimization layer will run on the broker and it will rewrite the extraction filter as clause of selector filters. +The optimization layer will run on the Broker and it will rewrite the extraction filter as clause of selector filters. For instance the following filter ```json diff --git a/docs/content/querying/groupbyquery.md b/docs/content/querying/groupbyquery.md index 1d502a9ac90..6ba1402c20a 100644 --- a/docs/content/querying/groupbyquery.md +++ b/docs/content/querying/groupbyquery.md @@ -226,22 +226,22 @@ Response for above query would look something like below... #### Strategies GroupBy queries can be executed using two different strategies. The default strategy for a cluster is determined by the -"druid.query.groupBy.defaultStrategy" runtime property on the broker. This can be overridden using "groupByStrategy" in +"druid.query.groupBy.defaultStrategy" runtime property on the Broker. This can be overridden using "groupByStrategy" in the query context. If neither the context field nor the property is set, the "v2" strategy will be used. - "v2", the default, is designed to offer better performance and memory management. This strategy generates per-segment results using a fully off-heap map. Data nodes merge the per-segment results using a fully off-heap concurrent facts map combined with an on-heap string dictionary. This may optionally involve spilling to disk. Data -nodes return sorted results to the broker, which merges result streams using an N-way merge. The broker materializes +nodes return sorted results to the Broker, which merges result streams using an N-way merge. The broker materializes the results if necessary (e.g. if the query sorts on columns other than its dimensions). Otherwise, it streams results back as they are merged. -- "v1", a legacy engine, generates per-segment results on data nodes (historical, realtime, middleManager) using a map which +- "v1", a legacy engine, generates per-segment results on data nodes (Historical, realtime, MiddleManager) using a map which is partially on-heap (dimension keys and the map itself) and partially off-heap (the aggregated values). Data nodes then merge the per-segment results using Druid's indexing mechanism. This merging is multi-threaded by default, but can -optionally be single-threaded. The broker merges the final result set using Druid's indexing mechanism again. The broker -merging is always single-threaded. Because the broker merges results using the indexing mechanism, it must materialize -the full result set before returning any results. On both the data nodes and the broker, the merging index is fully +optionally be single-threaded. The Broker merges the final result set using Druid's indexing mechanism again. The broker +merging is always single-threaded. Because the Broker merges results using the indexing mechanism, it must materialize +the full result set before returning any results. On both the data nodes and the Broker, the merging index is fully on-heap by default, but it can optionally store aggregated values off-heap. #### Differences between v1 and v2 @@ -257,9 +257,9 @@ that can complete successfully in one engine may exceed resource limits and fail - groupBy v1 imposes no limit on the number of concurrently running queries, whereas groupBy v2 controls memory usage by using a finite-sized merge buffer pool. By default, the number of merge buffers is 1/4 the number of processing threads. You can adjust this as necessary to balance concurrency and memory usage. -- groupBy v1 supports caching on either the broker or historical nodes, whereas groupBy v2 only supports caching on -historical nodes. -- groupBy v1 supports using [chunkPeriod](query-context.html) to parallelize merging on the broker, whereas groupBy v2 +- groupBy v1 supports caching on either the Broker or Historical nodes, whereas groupBy v2 only supports caching on +Historical nodes. +- groupBy v1 supports using [chunkPeriod](query-context.html) to parallelize merging on the Broker, whereas groupBy v2 ignores chunkPeriod. - groupBy v2 supports both array-based aggregation and hash-based aggregation. The array-based aggregation is used only when the grouping key is a single indexed string column. In array-based aggregation, the dictionary-encoded value is used @@ -304,7 +304,7 @@ concurrent query load. ##### Limit pushdown optimization -Druid pushes down the `limit` spec in groupBy queries to the segments on historicals wherever possible to early prune unnecessary intermediate results and minimize the amount of data transferred to brokers. By default, this technique is applied only when all fields in the `orderBy` spec is a subset of the grouping keys. This is because the `limitPushDown` doesn't guarantee the exact results if the `orderBy` spec includes any fields that are not in the grouping keys. However, you can enable this technique even in such cases if you can sacrifice some accuracy for fast query processing like in topN queries. See `forceLimitPushDown` in [advanced groupBy v2 configurations](#groupby-v2-configurations). +Druid pushes down the `limit` spec in groupBy queries to the segments on Historicals wherever possible to early prune unnecessary intermediate results and minimize the amount of data transferred to Brokers. By default, this technique is applied only when all fields in the `orderBy` spec is a subset of the grouping keys. This is because the `limitPushDown` doesn't guarantee the exact results if the `orderBy` spec includes any fields that are not in the grouping keys. However, you can enable this technique even in such cases if you can sacrifice some accuracy for fast query processing like in topN queries. See `forceLimitPushDown` in [advanced groupBy v2 configurations](#groupby-v2-configurations). ##### Optimizing hash table @@ -316,10 +316,10 @@ The default number of initial buckets is 1024 and the default max load factor of ##### Parallel combine -Once a historical finishes aggregation using the hash table, it sorts the aggregated results and merges them before sending to the -broker for N-way merge aggregation in the broker. By default, historicals use all their available processing threads +Once a Historical finishes aggregation using the hash table, it sorts the aggregated results and merges them before sending to the +Broker for N-way merge aggregation in the broker. By default, Historicals use all their available processing threads (configured by `druid.processing.numThreads`) for aggregation, but use a single thread for sorting and merging -aggregates which is an http thread to send data to brokers. +aggregates which is an http thread to send data to Brokers. This is to prevent some heavy groupBy queries from blocking other queries. In Druid, the processing threads are shared between all submitted queries and they are _not interruptible_. It means, if a heavy query takes all available @@ -338,7 +338,7 @@ aggregates from hash tables including spilled ones. Usually, leaf nodes are slow need to read data from disk. As a result, less threads are used for intermediate nodes by default. You can change the degree of intermediate nodes. See `intermediateCombineDegree` in [Advanced groupBy v2 configurations](#groupby-v2-configurations). -Please note that each historical needs two merge buffers to process a groupBy v2 query with parallel combine: one for +Please note that each Historical needs two merge buffers to process a groupBy v2 query with parallel combine: one for computing intermediate aggregates from each segment and another for combining intermediate aggregates in parallel. @@ -356,15 +356,15 @@ results acceptable. #### Nested groupBys -Nested groupBys (dataSource of type "query") are performed differently for "v1" and "v2". The broker first runs the +Nested groupBys (dataSource of type "query") are performed differently for "v1" and "v2". The Broker first runs the inner groupBy query in the usual way. "v1" strategy then materializes the inner query's results on-heap with Druid's indexing mechanism, and runs the outer query on these materialized results. "v2" strategy runs the outer query on the inner query's results stream with off-heap fact map and on-heap string dictionary that can spill to disk. Both -strategy perform the outer query on the broker in a single-threaded fashion. +strategy perform the outer query on the Broker in a single-threaded fashion. #### Configurations -This section describes the configurations for groupBy queries. You can set the runtime properties in the `runtime.properties` file on broker, historical, and MiddleManager nodes. You can set the query context parameters through the [query context](query-context.html). +This section describes the configurations for groupBy queries. You can set the runtime properties in the `runtime.properties` file on Broker, Historical, and MiddleManager nodes. You can set the query context parameters through the [query context](query-context.html). ##### Configurations for groupBy v2 @@ -424,7 +424,7 @@ Supported query contexts: |`intermediateCombineDegree`|Overrides the value of `druid.query.groupBy.intermediateCombineDegree`|None| |`numParallelCombineThreads`|Overrides the value of `druid.query.groupBy.numParallelCombineThreads`|None| |`sortByDimsFirst`|Sort the results first by dimension values and then by timestamp.|false| -|`forceLimitPushDown`|When all fields in the orderby are part of the grouping key, the broker will push limit application down to the historical nodes. When the sorting order uses fields that are not in the grouping key, applying this optimization can result in approximate results with unknown accuracy, so this optimization is disabled by default in that case. Enabling this context flag turns on limit push down for limit/orderbys that contain non-grouping key columns.|false| +|`forceLimitPushDown`|When all fields in the orderby are part of the grouping key, the Broker will push limit application down to the Historical nodes. When the sorting order uses fields that are not in the grouping key, applying this optimization can result in approximate results with unknown accuracy, so this optimization is disabled by default in that case. Enabling this context flag turns on limit push down for limit/orderbys that contain non-grouping key columns.|false| ##### GroupBy v1 configurations diff --git a/docs/content/querying/lookups.md b/docs/content/querying/lookups.md index 9a9feca365e..7c7ad81df2f 100644 --- a/docs/content/querying/lookups.md +++ b/docs/content/querying/lookups.md @@ -93,11 +93,11 @@ Dynamic Configuration Dynamic lookup configuration is an experimental feature. Static configuration is no longer supported. -The following documents the behavior of the cluster-wide config which is accessible through the coordinator. +The following documents the behavior of the cluster-wide config which is accessible through the Coordinator. The configuration is propagated through the concept of "tier" of servers. A "tier" is defined as a group of services which should receive a set of lookups. -For example, you might have all historicals be part of `__default`, and Peons be part of individual tiers for the datasources they are tasked with. -The tiers for lookups are completely independent of historical tiers. +For example, you might have all Historicals be part of `__default`, and Peons be part of individual tiers for the datasources they are tasked with. +The tiers for lookups are completely independent of Historical tiers. These configs are accessed using JSON through the following URI template @@ -117,9 +117,9 @@ These endpoints will return one of the following results: * 200 if the request succeeded (`GET` only) ## Configuration propagation behavior -The configuration is propagated to the query serving nodes (broker / router / peon / historical) by the coordinator. -The query serving nodes have an internal API for managing lookups on the node and those are used by the coordinator. -The coordinator periodically checks if any of the nodes need to load/drop lookups and updates them appropriately. +The configuration is propagated to the query serving processes (Broker / Router / Peon / Historical) by the Coordinator. +The query serving nodes have an internal API for managing lookups on the node and those are used by the Coordinator. +The Coordinator periodically checks if any of the nodes need to load/drop lookups and updates them appropriately. # API for configuring lookups @@ -288,7 +288,7 @@ To discover a list of tiers currently active in the cluster **instead of** ones A `GET` to `/druid/coordinator/v1/lookups/config/{tier}` will return a list of known lookup names for that tier. # Additional API related to status of configured lookups -These end points can be used to get the propagation status of configured lookups to lookup nodes such as historicals. +These end points can be used to get the propagation status of configured lookups to lookup nodes such as Historicals. ## List load status of all lookups `GET /druid/coordinator/v1/lookups/status` with optional query parameter `detailed`. @@ -352,16 +352,16 @@ The return value will be the json representation of the factory. ``` # Configuration -See [Lookups Dynamic Configuration](../configuration/index.html#lookups-dynamic-configuration) for coordinator configuration. +See [Lookups Dynamic Configuration](../configuration/index.html#lookups-dynamic-configuration) for Coordinator configuration. To configure a Broker / Router / Historical / Peon to announce itself as part of a lookup tier, use the `druid.zk.paths.lookupTier` property. |Property | Description | Default | |---------|-------------|---------| |`druid.lookup.lookupTier`| The tier for **lookups** for this node. This is independent of other tiers.|`__default`| -|`druid.lookup.lookupTierIsDatasource`|For some things like indexing service tasks, the datasource is passed in the runtime properties of a task. This option fetches the tierName from the same value as the datasource for the task. It is suggested to only use this as peon options for the indexing service, if at all. If true, `druid.lookup.lookupTier` MUST NOT be specified|`"false"`| +|`druid.lookup.lookupTierIsDatasource`|For some things like indexing service tasks, the datasource is passed in the runtime properties of a task. This option fetches the tierName from the same value as the datasource for the task. It is suggested to only use this as Peon options for the indexing service, if at all. If true, `druid.lookup.lookupTier` MUST NOT be specified|`"false"`| -To configure the behavior of the dynamic configuration manager, use the following properties on the coordinator: +To configure the behavior of the dynamic configuration manager, use the following properties on the Coordinator: |Property|Description|Default| |--------|-----------|-------| @@ -372,20 +372,20 @@ To configure the behavior of the dynamic configuration manager, use the followin ## Saving configuration across restarts -It is possible to save the configuration across restarts such that a node will not have to wait for coordinator action to re-populate its lookups. To do this the following property is set: +It is possible to save the configuration across restarts such that a node will not have to wait for Coordinator action to re-populate its lookups. To do this the following property is set: |Property|Description|Default| |--------|-----------|-------| |`druid.lookup.snapshotWorkingDir`|Working path used to store snapshot of current lookup configuration, leaving this property null will disable snapshot/bootstrap utility|null| -|`druid.lookup.enableLookupSyncOnStartup`|Enable the lookup synchronization process with coordinator on startup. The queryable nodes will fetch and load the lookups from the coordinator instead of waiting for the coordinator to load the lookups for them. Users may opt to disable this option if there are no lookups configured in the cluster.|true| +|`druid.lookup.enableLookupSyncOnStartup`|Enable the lookup synchronization process with Coordinator on startup. The queryable nodes will fetch and load the lookups from the Coordinator instead of waiting for the Coordinator to load the lookups for them. Users may opt to disable this option if there are no lookups configured in the cluster.|true| |`druid.lookup.numLookupLoadingThreads`|Number of threads for loading the lookups in parallel on startup. This thread pool is destroyed once startup is done. It is not kept during the lifetime of the JVM|Available Processors / 2| -|`druid.lookup.coordinatorFetchRetries`|How many times to retry to fetch the lookup bean list from coordinator, during the sync on startup.|3| +|`druid.lookup.coordinatorFetchRetries`|How many times to retry to fetch the lookup bean list from Coordinator, during the sync on startup.|3| |`druid.lookup.lookupStartRetries`|How many times to retry to start each lookup, either during the sync on startup, or during the runtime.|3| -|`druid.lookup.coordinatorRetryDelay`|How long to delay (in millis) between retries to fetch lookup list from the coordinator during the sync on startup.|60_000| +|`druid.lookup.coordinatorRetryDelay`|How long to delay (in millis) between retries to fetch lookup list from the Coordinator during the sync on startup.|60_000| ## Introspect a Lookup -The broker provides an API for lookup introspection if the lookup type implements a `LookupIntrospectHandler`. +The Broker provides an API for lookup introspection if the lookup type implements a `LookupIntrospectHandler`. A `GET` request to `/druid/v1/lookups/introspect/{lookupId}` will return the map of complete values. @@ -436,6 +436,6 @@ ex: `GET /druid/v1/lookups/introspect/nato-phonetic/values` ## Druid version 0.10.0 to 0.10.1 upgrade/downgrade Overall druid cluster lookups configuration is persisted in metadata store and also individual lookup nodes optionally persist a snapshot of loaded lookups on disk. If upgrading from druid version 0.10.0 to 0.10.1, then migration for all persisted metadata is handled automatically. -If downgrading from 0.10.1 to 0.9.0 then lookups updates done via coordinator while 0.10.1 was running, would be lost. +If downgrading from 0.10.1 to 0.9.0 then lookups updates done via Coordinator while 0.10.1 was running, would be lost. diff --git a/docs/content/querying/multitenancy.md b/docs/content/querying/multitenancy.md index 642d6ea0386..ea7408128f2 100644 --- a/docs/content/querying/multitenancy.md +++ b/docs/content/querying/multitenancy.md @@ -71,7 +71,7 @@ You can use this in concert with single-dimension partitioning to repartition yo ## Customizing data distribution -Druid additionally supports multitenancy by providing configurable means of distributing data. Druid's historical nodes +Druid additionally supports multitenancy by providing configurable means of distributing data. Druid's Historical nodes can be configured into [tiers](../operations/rule-configuration.html), and [rules](../operations/rule-configuration.html) can be set that determines which segments go into which tiers. One use case of this is that recent data tends to be accessed more frequently than older data. Tiering enables more recent segments to be hosted on more powerful hardware for better performance. @@ -95,5 +95,5 @@ Druid queries can optionally set a `priority` flag in the [query context](../que slow (download or reporting style queries) can be de-prioritized and more interactive queries can have higher priority. Broker nodes can also be dedicated to a given tier. For example, one set of broker nodes can be dedicated to fast interactive queries, -and a second set of broker nodes can be dedicated to slower reporting queries. Druid also provides a [router](../development/router.html) -node that can route queries to different brokers based on various query parameters (datasource, interval, etc.). +and a second set of Broker nodes can be dedicated to slower reporting queries. Druid also provides a [Router](../development/router.html) +node that can route queries to different Brokers based on various query parameters (datasource, interval, etc.). diff --git a/docs/content/querying/query-context.md b/docs/content/querying/query-context.md index 81b39d8f951..60fe1c71502 100644 --- a/docs/content/querying/query-context.md +++ b/docs/content/querying/query-context.md @@ -28,7 +28,7 @@ The query context is used for various query configuration parameters. The follow |property |default | description | |-----------------|----------------------------------------|----------------------| -|timeout | `druid.server.http.defaultQueryTimeout`| Query timeout in millis, beyond which unfinished queries will be cancelled. 0 timeout means `no timeout`. To set the default timeout, see [broker configuration](../configuration/index.html#broker) | +|timeout | `druid.server.http.defaultQueryTimeout`| Query timeout in millis, beyond which unfinished queries will be cancelled. 0 timeout means `no timeout`. To set the default timeout, see [Broker configuration](../configuration/index.html#broker) | |priority | `0` | Query Priority. Queries with higher priority get precedence for computational resources.| |queryId | auto-generated | Unique identifier given to this query. If a query ID is set or known, this can be used to cancel the query | |useCache | `true` | Flag indicating whether to leverage the query cache for this query. When set to false, it disables reading from the query cache for this query. When set to true, Druid uses druid.broker.cache.useCache or druid.historical.cache.useCache to determine whether or not to read from the query cache | @@ -37,11 +37,11 @@ The query context is used for various query configuration parameters. The follow |populateResultLevelCache | `false` | Flag indicating whether to save the results of the query to the result level cache. Primarily used for debugging. When set to false, it disables saving the results of this query to the query cache. When set to true, Druid uses druid.broker.cache.populateCache to determine whether or not to save the results of this query to the query cache | |bySegment | `false` | Return "by segment" results. Primarily used for debugging, setting it to `true` returns results associated with the data segment they came from | |finalize | `true` | Flag indicating whether to "finalize" aggregation results. Primarily used for debugging. For instance, the `hyperUnique` aggregator will return the full HyperLogLog sketch instead of the estimated cardinality when this flag is set to `false` | -|chunkPeriod | `P0D` (off) | At the broker node level, long interval queries (of any type) may be broken into shorter interval queries to parallelize merging more than normal. Broken up queries will use a larger share of cluster resources, but, if you use groupBy "v1, it may be able to complete faster as a result. Use ISO 8601 periods. For example, if this property is set to `P1M` (one month), then a query covering a year would be broken into 12 smaller queries. The broker uses its query processing executor service to initiate processing for query chunks, so make sure "druid.processing.numThreads" is configured appropriately on the broker. [groupBy queries](groupbyquery.html) do not support chunkPeriod by default, although they do if using the legacy "v1" engine. This context is deprecated since it's only useful for groupBy "v1", and will be removed in the future releases.| -|maxScatterGatherBytes| `druid.server.http.maxScatterGatherBytes` | Maximum number of bytes gathered from data nodes such as historicals and realtime processes to execute a query. This parameter can be used to further reduce `maxScatterGatherBytes` limit at query time. See [broker configuration](../configuration/index.html#broker) for more details.| +|chunkPeriod | `P0D` (off) | At the Broker node level, long interval queries (of any type) may be broken into shorter interval queries to parallelize merging more than normal. Broken up queries will use a larger share of cluster resources, but, if you use groupBy "v1, it may be able to complete faster as a result. Use ISO 8601 periods. For example, if this property is set to `P1M` (one month), then a query covering a year would be broken into 12 smaller queries. The broker uses its query processing executor service to initiate processing for query chunks, so make sure "druid.processing.numThreads" is configured appropriately on the broker. [groupBy queries](groupbyquery.html) do not support chunkPeriod by default, although they do if using the legacy "v1" engine. This context is deprecated since it's only useful for groupBy "v1", and will be removed in the future releases.| +|maxScatterGatherBytes| `druid.server.http.maxScatterGatherBytes` | Maximum number of bytes gathered from data nodes such as Historicals and realtime processes to execute a query. This parameter can be used to further reduce `maxScatterGatherBytes` limit at query time. See [Broker configuration](../configuration/index.html#broker) for more details.| |maxQueuedBytes | `druid.broker.http.maxQueuedBytes` | Maximum number of bytes queued per query before exerting backpressure on the channel to the data server. Similar to `maxScatterGatherBytes`, except unlike that configuration, this one will trigger backpressure rather than query failure. Zero means disabled.| -|serializeDateTimeAsLong| `false` | If true, DateTime is serialized as long in the result returned by broker and the data transportation between broker and compute node| -|serializeDateTimeAsLongInner| `false` | If true, DateTime is serialized as long in the data transportation between broker and compute node| +|serializeDateTimeAsLong| `false` | If true, DateTime is serialized as long in the result returned by Broker and the data transportation between Broker and compute node| +|serializeDateTimeAsLongInner| `false` | If true, DateTime is serialized as long in the data transportation between Broker and compute node| In addition, some query types offer context parameters specific to that query type. diff --git a/docs/content/querying/querying.md b/docs/content/querying/querying.md index 1aa9ca50106..c26e6011cb3 100644 --- a/docs/content/querying/querying.md +++ b/docs/content/querying/querying.md @@ -27,7 +27,7 @@ title: "Querying" Queries are made using an HTTP REST style request to queryable nodes ([Broker](../design/broker.html), [Historical](../design/historical.html). [Peons](../design/peons.html)) that are running stream ingestion tasks can also accept queries. The query is expressed in JSON and each of these node types expose the same -REST query interface. For normal Druid operations, queries should be issued to the broker nodes. Queries can be posted +REST query interface. For normal Druid operations, queries should be issued to the Broker nodes. Queries can be posted to the queryable nodes like this - ```bash @@ -79,7 +79,7 @@ Where possible, we recommend using [Timeseries]() and [TopN]() queries instead o Queries can be cancelled explicitly using their unique identifier. If the query identifier is set at the time of query, or is otherwise known, the following -endpoint can be used on the broker or router to cancel the query. +endpoint can be used on the Broker or Router to cancel the query. ```sh DELETE /druid/v2/{queryId} diff --git a/docs/content/querying/scan-query.md b/docs/content/querying/scan-query.md index 3963800ae7d..cb4791f7ec0 100644 --- a/docs/content/querying/scan-query.md +++ b/docs/content/querying/scan-query.md @@ -178,7 +178,7 @@ The format of the result when resultFormat equals to `compactedList`: The biggest difference between select query and scan query is that, scan query doesn't retain all rows in memory before rows can be returned to client. It will cause memory pressure if too many rows required by select query. Scan query doesn't have this issue. -Scan query can return all rows without issuing another pagination query, which is extremely useful when query against historical or realtime node directly. +Scan query can return all rows without issuing another pagination query, which is extremely useful when query against Historical or realtime node directly. ## Legacy mode diff --git a/docs/content/querying/searchquery.md b/docs/content/querying/searchquery.md index 8a9c5b17a24..3727a5d0169 100644 --- a/docs/content/querying/searchquery.md +++ b/docs/content/querying/searchquery.md @@ -56,7 +56,7 @@ There are several main parts to a search query: |dataSource|A String or Object defining the data source to query, very similar to a table in a relational database. See [DataSource](../querying/datasource.html) for more information.|yes| |granularity|Defines the granularity of the query. See [Granularities](../querying/granularities.html).|yes| |filter|See [Filters](../querying/filters.html).|no| -|limit| Defines the maximum number per historical node (parsed as int) of search results to return. |no (default to 1000)| +|limit| Defines the maximum number per Historical node (parsed as int) of search results to return. |no (default to 1000)| |intervals|A JSON Object representing ISO-8601 Intervals. This defines the time ranges to run the query over.|yes| |searchDimensions|The dimensions to run the search over. Excluding this means the search is run over all dimensions.|no| |query|See [SearchQuerySpec](../querying/searchqueryspec.html).|yes| @@ -105,7 +105,7 @@ The format of the result is: #### Strategies Search queries can be executed using two different strategies. The default strategy is determined by the -"druid.query.search.searchStrategy" runtime property on the broker. This can be overridden using "searchStrategy" in the +"druid.query.search.searchStrategy" runtime property on the Broker. This can be overridden using "searchStrategy" in the query context. If neither the context field nor the property is set, the "useIndexes" strategy will be used. - "useIndexes" strategy, the default, first categorizes search dimensions into two groups according to their support for diff --git a/docs/content/querying/segmentmetadataquery.md b/docs/content/querying/segmentmetadataquery.md index ae6d6b83201..aa7b9f19d25 100644 --- a/docs/content/querying/segmentmetadataquery.md +++ b/docs/content/querying/segmentmetadataquery.md @@ -93,7 +93,7 @@ Only columns which are dimensions (ie, have type `STRING`) will have any cardina If an interval is not specified, the query will use a default interval that spans a configurable period before the end time of the most recent segment. -The length of this default time period is set in the broker configuration via: +The length of this default time period is set in the Broker configuration via: druid.query.segmentMetadata.defaultHistory ### toInclude @@ -130,7 +130,7 @@ This is a list of properties that determines the amount of information returned By default, the "cardinality", "interval", and "minmax" types will be used. If a property is not needed, omitting it from this list will result in a more efficient query. -The default analysis types can be set in the broker configuration via: +The default analysis types can be set in the Broker configuration via: `druid.query.segmentMetadata.defaultAnalysisTypes` Types of column analyses are described below: diff --git a/docs/content/querying/sql.md b/docs/content/querying/sql.md index 4a0c0d59055..257963dcec5 100644 --- a/docs/content/querying/sql.md +++ b/docs/content/querying/sql.md @@ -31,12 +31,12 @@ subject to change. Druid SQL is a built-in SQL layer and an alternative to Druid's native JSON-based query language, and is powered by a parser and planner based on [Apache Calcite](https://calcite.apache.org/). Druid SQL translates SQL into native Druid -queries on the query broker (the first node you query), which are then passed down to data nodes as native Druid -queries. Other than the (slight) overhead of translating SQL on the broker, there isn't an additional performance +queries on the query Broker (the first node you query), which are then passed down to data nodes as native Druid +queries. Other than the (slight) overhead of translating SQL on the Broker, there isn't an additional performance penalty versus native queries. To enable Druid SQL, make sure you have set `druid.sql.enable = true` either in your common.runtime.properties or your -broker's runtime.properties. +Broker's runtime.properties. ## Query syntax @@ -322,18 +322,18 @@ computed in memory. See the TopN documentation for more details. - [GroupBy](groupbyquery.html) is used for all other aggregations, including any nested aggregation queries. Druid's GroupBy is a traditional aggregation engine: it delivers exact results and rankings and supports a wide variety of features. GroupBy aggregates in memory if it can, but it may spill to disk if it doesn't have enough memory to complete -your query. Results are streamed back from data nodes through the broker if you ORDER BY the same expressions in your +your query. Results are streamed back from data nodes through the Broker if you ORDER BY the same expressions in your GROUP BY clause, or if you don't have an ORDER BY at all. If your query has an ORDER BY referencing expressions that -don't appear in the GROUP BY clause (like aggregation functions) then the broker will materialize a list of results in +don't appear in the GROUP BY clause (like aggregation functions) then the Broker will materialize a list of results in memory, up to a max of your LIMIT, if any. See the GroupBy documentation for details about tuning performance and memory use. If your query does nested aggregations (an aggregation subquery in your FROM clause) then Druid will execute it as a [nested GroupBy](groupbyquery.html#nested-groupbys). In nested GroupBys, the innermost aggregation is distributed, but -all outer aggregations beyond that take place locally on the query broker. +all outer aggregations beyond that take place locally on the query Broker. Semi-join queries containing WHERE clauses like `col IN (SELECT expr FROM ...)` are executed with a special process. The -broker will first translate the subquery into a GroupBy to find distinct values of `expr`. Then, the broker will rewrite +Broker will first translate the subquery into a GroupBy to find distinct values of `expr`. Then, the broker will rewrite the subquery to a literal filter, like `col IN (val1, val2, ...)` and run the outer query. The configuration parameter druid.sql.planner.maxSemiJoinRowsInMemory controls the maximum number of values that will be materialized for this kind of plan. @@ -351,10 +351,10 @@ Druid SQL will use approximate algorithms in some situations: - The `COUNT(DISTINCT col)` aggregation functions by default uses a variant of [HyperLogLog](http://algo.inria.fr/flajolet/Publications/FlFuGaMe07.pdf), a fast approximate distinct counting algorithm. Druid SQL will switch to exact distinct counts if you set "useApproximateCountDistinct" to "false", either -through query context or through broker configuration. +through query context or through Broker configuration. - GROUP BY queries over a single column with ORDER BY and LIMIT may be executed using the TopN engine, which uses an approximate algorithm. Druid SQL will switch to an exact grouping algorithm if you set "useApproximateTopN" to "false", -either through query context or through broker configuration. +either through query context or through Broker configuration. - The APPROX_COUNT_DISTINCT and APPROX_QUANTILE aggregation functions always use approximate algorithms, regardless of configuration. @@ -444,7 +444,7 @@ you've downloaded the Avatica client jar, add it to your classpath and use the c Example code: ```java -// Connect to /druid/v2/sql/avatica/ on your broker. +// Connect to /druid/v2/sql/avatica/ on your Broker. String url = "jdbc:avatica:remote:url=http://localhost:8082/druid/v2/sql/avatica/"; // Set any connection context parameters you need here (see "Connection context" below). @@ -469,8 +469,8 @@ so avoid those. #### Connection stickiness -Druid's JDBC server does not share connection state between brokers. This means that if you're using JDBC and have -multiple Druid brokers, you should either connect to a specific broker, or use a load balancer with sticky sessions +Druid's JDBC server does not share connection state between Brokers. This means that if you're using JDBC and have +multiple Druid Brokers, you should either connect to a specific Broker, or use a load balancer with sticky sessions enabled. The Druid Router node provides connection stickiness when balancing JDBC requests, and can be used to achieve the necessary stickiness even with a normal non-sticky load balancer. Please see the [Router](../development/router.html) documentation for more details. @@ -491,15 +491,15 @@ Connection context can be specified as JDBC connection properties or as a "conte |Parameter|Description|Default value| |---------|-----------|-------------| |`sqlQueryId`|Unique identifier given to this SQL query. For HTTP client, it will be returned in `X-Druid-SQL-Query-Id` header.|auto-generated| -|`sqlTimeZone`|Sets the time zone for this connection, which will affect how time functions and timestamp literals behave. Should be a time zone name like "America/Los_Angeles" or offset like "-08:00".|druid.sql.planner.sqlTimeZone on the broker (default: UTC)| -|`useApproximateCountDistinct`|Whether to use an approximate cardinalty algorithm for `COUNT(DISTINCT foo)`.|druid.sql.planner.useApproximateCountDistinct on the broker (default: true)| -|`useApproximateTopN`|Whether to use approximate [TopN queries](topnquery.html) when a SQL query could be expressed as such. If false, exact [GroupBy queries](groupbyquery.html) will be used instead.|druid.sql.planner.useApproximateTopN on the broker (default: true)| -|`useFallback`|Whether to evaluate operations on the broker when they cannot be expressed as Druid queries. This option is not recommended for production since it can generate unscalable query plans. If false, SQL queries that cannot be translated to Druid queries will fail.|druid.sql.planner.useFallback on the broker (default: false)| +|`sqlTimeZone`|Sets the time zone for this connection, which will affect how time functions and timestamp literals behave. Should be a time zone name like "America/Los_Angeles" or offset like "-08:00".|druid.sql.planner.sqlTimeZone on the Broker (default: UTC)| +|`useApproximateCountDistinct`|Whether to use an approximate cardinalty algorithm for `COUNT(DISTINCT foo)`.|druid.sql.planner.useApproximateCountDistinct on the Broker (default: true)| +|`useApproximateTopN`|Whether to use approximate [TopN queries](topnquery.html) when a SQL query could be expressed as such. If false, exact [GroupBy queries](groupbyquery.html) will be used instead.|druid.sql.planner.useApproximateTopN on the Broker (default: true)| +|`useFallback`|Whether to evaluate operations on the Broker when they cannot be expressed as Druid queries. This option is not recommended for production since it can generate unscalable query plans. If false, SQL queries that cannot be translated to Druid queries will fail.|druid.sql.planner.useFallback on the Broker (default: false)| ### Retrieving metadata -Druid brokers infer table and column metadata for each dataSource from segments loaded in the cluster, and use this to -plan SQL queries. This metadata is cached on broker startup and also updated periodically in the background through +Druid Brokers infer table and column metadata for each dataSource from segments loaded in the cluster, and use this to +plan SQL queries. This metadata is cached on Broker startup and also updated periodically in the background through [SegmentMetadata queries](segmentmetadataquery.html). Background metadata refreshing is triggered by segments entering and exiting the cluster, and can also be throttled through configuration. @@ -581,14 +581,14 @@ Segments table provides details on all Druid segments, whether they are publishe |version|Version string (generally an ISO8601 timestamp corresponding to when the segment set was first started). Higher version means the more recently created segment. Version comparing is based on string comparison.| |partition_num|Partition number (an integer, unique within a datasource+interval+version; may not necessarily be contiguous)| |num_replicas|Number of replicas of this segment currently being served| -|num_rows|Number of rows in current segment, this value could be null if unkown to broker at query time| +|num_rows|Number of rows in current segment, this value could be null if unkown to Broker at query time| |is_published|Boolean is represented as long type where 1 = true, 0 = false. 1 represents this segment has been published to the metadata store| -|is_available|Boolean is represented as long type where 1 = true, 0 = false. 1 if this segment is currently being served by any server(historical or realtime)| +|is_available|Boolean is represented as long type where 1 = true, 0 = false. 1 if this segment is currently being served by any server(Historical or realtime)| |is_realtime|Boolean is represented as long type where 1 = true, 0 = false. 1 if this segment is being served on any type of realtime tasks| |payload|JSON-serialized data segment payload| ### SERVERS table -Servers table lists all data servers(any server that hosts a segment). It includes both historicals and peons. +Servers table lists all data servers(any server that hosts a segment). It includes both Historicals and Peons. |Column|Notes| |------|-----| @@ -596,7 +596,7 @@ Servers table lists all data servers(any server that hosts a segment). It includ |host|Hostname of the server| |plaintext_port|Unsecured port of the server, or -1 if plaintext traffic is disabled| |tls_port|TLS port of the server, or -1 if TLS is disabled| -|server_type|Type of Druid service. Possible values include: historical, realtime and indexer_executor(peon).| +|server_type|Type of Druid service. Possible values include: Historical, realtime and indexer_executor(Peon).| |tier|Distribution tier see [druid.server.tier](#../configuration/index.html#Historical-General-Configuration)| |current_size|Current size of segments in bytes on this server| |max_size|Max size in bytes this server recommends to assign to segments see [druid.server.maxSize](#../configuration/index.html#Historical-General-Configuration)| @@ -638,7 +638,7 @@ check out [ingestion tasks](#../ingestion/tasks.html) |type|Task type, for example this value is "index" for indexing tasks. See [tasks-overview](../ingestion/tasks.md)| |datasource|Datasource name being indexed| |created_time|Timestamp in ISO8601 format corresponding to when the ingestion task was created. Note that this value is populated for completed and waiting tasks. For running and pending tasks this value is set to 1970-01-01T00:00:00Z| -|queue_insertion_time|Timestamp in ISO8601 format corresponding to when this task was added to the queue on the overlord| +|queue_insertion_time|Timestamp in ISO8601 format corresponding to when this task was added to the queue on the Overlord| |status|Status of a task can be RUNNING, FAILED, SUCCESS| |runner_status|Runner status of a completed task would be NONE, for in-progress tasks this can be RUNNING, WAITING, PENDING| |duration|Time it took to finish the task in milliseconds, this value is present only for completed tasks| @@ -656,7 +656,7 @@ SELECT * FROM sys.tasks where status='FAILED'; ## Server configuration -The Druid SQL server is configured through the following properties on the broker. +The Druid SQL server is configured through the following properties on the Broker. |Property|Description|Default| |--------|-----------|-------| @@ -674,7 +674,7 @@ The Druid SQL server is configured through the following properties on the broke |`druid.sql.planner.selectThreshold`|Page size threshold for [Select queries](../querying/select-query.html). Select queries for larger resultsets will be issued back-to-back using pagination.|1000| |`druid.sql.planner.useApproximateCountDistinct`|Whether to use an approximate cardinalty algorithm for `COUNT(DISTINCT foo)`.|true| |`druid.sql.planner.useApproximateTopN`|Whether to use approximate [TopN queries](../querying/topnquery.html) when a SQL query could be expressed as such. If false, exact [GroupBy queries](../querying/groupbyquery.html) will be used instead.|true| -|`druid.sql.planner.useFallback`|Whether to evaluate operations on the broker when they cannot be expressed as Druid queries. This option is not recommended for production since it can generate unscalable query plans. If false, SQL queries that cannot be translated to Druid queries will fail.|false| +|`druid.sql.planner.useFallback`|Whether to evaluate operations on the Broker when they cannot be expressed as Druid queries. This option is not recommended for production since it can generate unscalable query plans. If false, SQL queries that cannot be translated to Druid queries will fail.|false| |`druid.sql.planner.requireTimeCondition`|Whether to require SQL to have filter conditions on __time column so that all generated native queries will have user specified intervals. If true, all queries wihout filter condition on __time column will fail|false| |`druid.sql.planner.sqlTimeZone`|Sets the default time zone for the server, which will affect how time functions and timestamp literals behave. Should be a time zone name like "America/Los_Angeles" or offset like "-08:00".|UTC| diff --git a/docs/content/querying/topnquery.md b/docs/content/querying/topnquery.md index 8963f7bb119..db662a8db09 100644 --- a/docs/content/querying/topnquery.md +++ b/docs/content/querying/topnquery.md @@ -26,7 +26,7 @@ title: "TopN queries" TopN queries return a sorted set of results for the values in a given dimension according to some criteria. Conceptually, they can be thought of as an approximate [GroupByQuery](../querying/groupbyquery.html) over a single dimension with an [Ordering](../querying/limitspec.html) spec. TopNs are much faster and resource efficient than GroupBys for this use case. These types of queries take a topN query object and return an array of JSON objects where each object represents a value asked for by the topN query. -TopNs are approximate in that each node will rank their top K results and only return those top K results to the broker. K, by default in Druid, is `max(1000, threshold)`. In practice, this means that if you ask for the top 1000 items ordered, the correctness of the first ~900 items will be 100%, and the ordering of the results after that is not guaranteed. TopNs can be made more accurate by increasing the threshold. +TopNs are approximate in that each node will rank their top K results and only return those top K results to the Broker. K, by default in Druid, is `max(1000, threshold)`. In practice, this means that if you ask for the top 1000 items ordered, the correctness of the first ~900 items will be 100%, and the ordering of the results after that is not guaranteed. TopNs can be made more accurate by increasing the threshold. A topN query object looks like: diff --git a/docs/content/toc.md b/docs/content/toc.md index f6e080047ba..5d67c19a570 100644 --- a/docs/content/toc.md +++ b/docs/content/toc.md @@ -101,13 +101,12 @@ layout: toc * [Overview](/docs/VERSION/design/index.html) * Storage * [Segments](/docs/VERSION/design/segments.html) - * Node Types - * [Historical](/docs/VERSION/design/historical.html) - * [Broker](/docs/VERSION/design/broker.html) + * [Processes and Servers](/docs/VERSION/design/processes.html) * [Coordinator](/docs/VERSION/design/coordinator.html) - * [Indexing Service](/docs/VERSION/design/indexing-service.html) - * [Overlord](/docs/VERSION/design/overlord.html) - * [MiddleManager](/docs/VERSION/design/middlemanager.html) + * [Overlord](/docs/VERSION/design/overlord.html) + * [Broker](/docs/VERSION/design/broker.html) + * [Historical](/docs/VERSION/design/historical.html) + * [MiddleManager](/docs/VERSION/design/middlemanager.html) * [Peons](/docs/VERSION/design/peons.html) * [Realtime (Deprecated)](/docs/VERSION/design/realtime.html) * Dependencies diff --git a/docs/content/tutorials/cluster.md b/docs/content/tutorials/cluster.md index c4c2b193f25..579ae1d048a 100644 --- a/docs/content/tutorials/cluster.md +++ b/docs/content/tutorials/cluster.md @@ -27,12 +27,19 @@ title: "Clustering" Druid is designed to be deployed as a scalable, fault-tolerant cluster. In this document, we'll set up a simple cluster and discuss how it can be further configured to meet -your needs. This simple cluster will feature scalable, fault-tolerant servers for Historicals and MiddleManagers, and a single -coordination server to host the Coordinator and Overlord processes. In production, we recommend deploying Coordinators and Overlords in a fault-tolerant -configuration as well. +your needs. + +This simple cluster will feature: + - A single Master server to host the Coordinator and Overlord processes + - Scalable, fault-tolerant Data servers running Historical and MiddleManager processes + - Query servers, hosting Druid Broker processes + +In production, we recommend deploying multiple Master servers with Coordinator and Overlord processes in a fault-tolerant configuration as well. ## Select hardware +### Master Server + The Coordinator and Overlord processes can be co-located on a single server that is responsible for handling the metadata and coordination needs of your cluster. The equivalent of an AWS [m3.xlarge](https://aws.amazon.com/ec2/instance-types/#M3) is sufficient for most clusters. This hardware offers: @@ -41,6 +48,8 @@ hardware offers: - 15 GB RAM - 80 GB SSD storage +### Data Server + Historicals and MiddleManagers can be colocated on a single server to handle the actual data in your cluster. These servers benefit greatly from CPU, RAM, and SSDs. The equivalent of an AWS [r3.2xlarge](https://aws.amazon.com/ec2/instance-types/#r3) is a good starting point. This hardware offers: @@ -49,6 +58,8 @@ good starting point. This hardware offers: - 61 GB RAM - 160 GB SSD storage +### Query Server + Druid Brokers accept queries and farm them out to the rest of the cluster. They also optionally maintain an in-memory query cache. These servers benefit greatly from CPU and RAM, and can also be deployed on the equivalent of an AWS [r3.2xlarge](https://aws.amazon.com/ec2/instance-types/#r3). This hardware @@ -191,7 +202,7 @@ using this functionality, then at this point you should If you will be loading data from a Hadoop cluster, then at this point you should configure Druid to be aware of your cluster: -- Update `druid.indexer.task.hadoopWorkingPath` in `conf/middleManager/runtime.properties` to +- Update `druid.indexer.task.hadoopWorkingPath` in `conf/druid/middleManager/runtime.properties` to a path on HDFS that you'd like to use for temporary files required during the indexing process. `druid.indexer.task.hadoopWorkingPath=/tmp/druid-indexing` is a common choice. @@ -207,8 +218,13 @@ For more info, please see [batch ingestion](../ingestion/batch-ingestion.html). ## Configure addresses for Druid coordination -In this simple cluster, you will deploy a single Druid Coordinator, a -single Druid Overlord, a single ZooKeeper instance, and an embedded Derby metadata store on the same server. +In this simple cluster, you will deploy a single Master server containing the following: +- A single Druid Coordinator process +- A single Druid Overlord process +- A single ZooKeeper istance +- An embedded Derby metadata store + +The processes on the cluster need to be configured with the addresses of this ZK instance and the metadata store. In `conf/druid/_common/common.runtime.properties`, replace "zk.service.host" with the address of the machine that runs your ZK instance: @@ -222,13 +238,13 @@ In `conf/druid/_common/common.runtime.properties`, replace - `druid.metadata.storage.connector.host`
-In production, we recommend running 2 servers, each running a Druid Coordinator -and a Druid Overlord. We also recommend running a ZooKeeper cluster on its own dedicated hardware, +In production, we recommend running 2 Master servers, each running a Druid Coordinator process +and a Druid Overlord process. We also recommend running a ZooKeeper cluster on its own dedicated hardware, as well as replicated metadata storage such as MySQL or PostgreSQL, on its own dedicated hardware.
-## Tune Druid processes that serve queries +## Tune processes on the Data Server Druid Historicals and MiddleManagers can be co-located on the same hardware. Both Druid processes benefit greatly from being tuned to the hardware they run on. If you are running Tranquility Server or Kafka, you can also colocate Tranquility with these two Druid processes. @@ -255,7 +271,7 @@ Keep -XX:MaxDirectMemory >= numThreads*sizeBytes, otherwise Druid will fail to s Please see the Druid [configuration documentation](../configuration/index.html) for a full description of all possible configuration options. -## Tune Druid Brokers +## Tune Druid Brokers on the Query Server Druid Brokers also benefit greatly from being tuned to the hardware they run on. If you are using [r3.2xlarge](https://aws.amazon.com/ec2/instance-types/#r3) EC2 instances, @@ -284,27 +300,34 @@ possible configuration options. If you're using a firewall or some other system that only allows traffic on specific ports, allow inbound connections on the following: -- 1527 (Derby on your Coordinator; not needed if you are using a separate metadata store like MySQL or PostgreSQL) +### Master Server +- 1527 (Derby metadata store; not needed if you are using a separate metadata store like MySQL or PostgreSQL) - 2181 (ZooKeeper; not needed if you are using a separate ZooKeeper cluster) - 8081 (Coordinator) -- 8082 (Broker) -- 8083 (Historical) -- 8084 (Standalone Realtime, if used) -- 8088 (Router, if used) - 8090 (Overlord) + +### Data Server +- 8083 (Historical) - 8091, 8100–8199 (Druid Middle Manager; you may need higher than port 8199 if you have a very high `druid.worker.capacity`) + +### Query Server +- 8082 (Broker) +- 8088 (Router, if used) + +### Other - 8200 (Tranquility Server, if used) +- 8084 (Standalone Realtime, if used, deprecated)
In production, we recommend deploying ZooKeeper and your metadata store on their own dedicated hardware, -rather than on the Coordinator server. +rather than on the Master server.
-## Start Coordinator, Overlord, Zookeeper, and metadata store +## Start Master Server -Copy the Druid distribution and your edited configurations to your coordination -server. If you have been editing the configurations on your local machine, you can use *rsync* to -copy them: +Copy the Druid distribution and your edited configurations to your Master server. + +If you have been editing the configurations on your local machine, you can use *rsync* to copy them: ```bash rsync -az apache-druid-#{DRUIDVERSION}/ COORDINATION_SERVER:apache-druid-#{DRUIDVERSION}/ @@ -334,18 +357,18 @@ java `cat conf/druid/overlord/jvm.config | xargs` -cp conf/druid/_common:conf/dr You should see a log message printed out for each service that starts up. You can view detailed logs for any service by looking in the `var/log/druid` directory using another terminal. -## Start Historicals and MiddleManagers +## Start Data Server -Copy the Druid distribution and your edited configurations to your servers set aside for the Druid Historicals and MiddleManagers. +Copy the Druid distribution and your edited configurations to your Data servers set aside for the Druid Historicals and MiddleManagers. -On each one, *cd* into the distribution and run this command to start a Data server: +On each one, *cd* into the distribution and run this command to start the Data server processes: ```bash java `cat conf/druid/historical/jvm.config | xargs` -cp conf/druid/_common:conf/druid/historical:lib/* org.apache.druid.cli.Main server historical java `cat conf/druid/middleManager/jvm.config | xargs` -cp conf/druid/_common:conf/druid/middleManager:lib/* org.apache.druid.cli.Main server middleManager ``` -You can add more servers with Druid Historicals and MiddleManagers as needed. +You can add more Data servers with Druid Historicals and MiddleManagers as needed.
For clusters with complex resource allocation needs, you can break apart Historicals and MiddleManagers and scale the components individually. @@ -365,17 +388,17 @@ cd tranquility-distribution-0.8.0 bin/tranquility -configFile /conf/tranquility/.json ``` -## Start Druid Broker +## Start Query Server -Copy the Druid distribution and your edited configurations to your servers set aside for the Druid Brokers. +Copy the Druid distribution and your edited configurations to your Query servers set aside for the Druid Brokers. -On each one, *cd* into the distribution and run this command to start a Broker (you may want to pipe the output to a log file): +On each Query server, *cd* into the distribution and run this command to start the Broker process (you may want to pipe the output to a log file): ```bash java `cat conf/druid/broker/jvm.config | xargs` -cp conf/druid/_common:conf/druid/broker:lib/* org.apache.druid.cli.Main server broker ``` -You can add more Brokers as needed based on query load. +You can add more Query servers as needed based on query load. ## Loading data diff --git a/docs/content/tutorials/tutorial-batch.md b/docs/content/tutorials/tutorial-batch.md index 972569f49d1..41f62aa5a03 100644 --- a/docs/content/tutorials/tutorial-batch.md +++ b/docs/content/tutorials/tutorial-batch.md @@ -34,7 +34,7 @@ don't need to have loaded any data yet. ## Preparing the data and the ingestion task spec -A data load is initiated by submitting an *ingestion task* spec to the Druid overlord. For this tutorial, we'll be loading the sample Wikipedia page edits data. +A data load is initiated by submitting an *ingestion task* spec to the Druid Overlord. For this tutorial, we'll be loading the sample Wikipedia page edits data. The Druid package includes the following sample native batch ingestion task spec at `quickstart/tutorial/wikipedia-index.json`, shown here for convenience, which has been configured to read the `quickstart/tutorial/wikiticker-2015-09-12-sampled.json.gz` input file: @@ -117,7 +117,7 @@ a task that loads the `wikiticker-2015-09-12-sampled.json.gz` file included in t For convenience, the Druid package includes a batch ingestion helper script at `bin/post-index-task`. -This script will POST an ingestion task to the Druid overlord and poll Druid until the data is available for querying. +This script will POST an ingestion task to the Druid Overlord and poll Druid until the data is available for querying. Run the following command from Druid package root: @@ -163,13 +163,13 @@ Which will print the ID of the task if the submission was successful: {"task":"index_wikipedia_2018-06-09T21:30:32.802Z"} ``` -To view the status of the ingestion task, go to the overlord console: +To view the status of the ingestion task, go to the Overlord console: [http://localhost:8090/console.html](http://localhost:8090/console.html). You can refresh the console periodically, and after the task is successful, you should see a "SUCCESS" status for the task. -After the ingestion task finishes, the data will be loaded by historical nodes and available for +After the ingestion task finishes, the data will be loaded by Historical nodes and available for querying within a minute or two. You can monitor the progress of loading the data in the -coordinator console, by checking whether there is a datasource "wikipedia" with a blue circle +Coordinator console, by checking whether there is a datasource "wikipedia" with a blue circle indicating "fully available": [http://localhost:8081/#/](http://localhost:8081/#/). ![Coordinator console](../tutorials/img/tutorial-batch-01.png "Wikipedia 100% loaded") diff --git a/docs/content/tutorials/tutorial-compaction.md b/docs/content/tutorials/tutorial-compaction.md index daf1f3c851f..201ea0090fc 100644 --- a/docs/content/tutorials/tutorial-compaction.md +++ b/docs/content/tutorials/tutorial-compaction.md @@ -97,11 +97,11 @@ After the task finishes, refresh the http://localhost:8081/#/datasources/compact The original 24 segments will eventually be marked as "unused" by the Coordinator and removed, with the new compacted segment remaining. -By default, the Druid coordinator will not mark segments as unused until the coordinator process has been up for at least 15 minutes, so you may see the old segment set and the new compacted set at the same time in the coordinator, e.g.: +By default, the Druid Coordinator will not mark segments as unused until the Coordinator process has been up for at least 15 minutes, so you may see the old segment set and the new compacted set at the same time in the Coordinator, e.g.: ![Compacted segments intermediate state](../tutorials/img/tutorial-compaction-01.png "Compacted segments intermediate state") -The new compacted segment has a more recent version than the original segments, so even when both sets of segments are shown by the coordinator, queries will only read from the new compacted segment. +The new compacted segment has a more recent version than the original segments, so even when both sets of segments are shown by the Coordinator, queries will only read from the new compacted segment. Let's try running a COUNT(*) on `compaction-tutorial` again, where the row count should still be 39,244: @@ -115,7 +115,7 @@ dsql> select count(*) from "compaction-tutorial"; Retrieved 1 row in 1.30s. ``` -After the coordinator has been running for at least 15 minutes, the http://localhost:8081/#/datasources/compaction-tutorial page should show there is only 1 segment: +After the Coordinator has been running for at least 15 minutes, the http://localhost:8081/#/datasources/compaction-tutorial page should show there is only 1 segment: ![Compacted segments final state](../tutorials/img/tutorial-compaction-02.png "Compacted segments final state") diff --git a/docs/content/tutorials/tutorial-delete-data.md b/docs/content/tutorials/tutorial-delete-data.md index 146658d2fef..6eba8f05804 100644 --- a/docs/content/tutorials/tutorial-delete-data.md +++ b/docs/content/tutorials/tutorial-delete-data.md @@ -108,7 +108,7 @@ On http://localhost:8081/#/datasources/deletion-tutorial, click one of the remai The top of the info box shows the full segment ID, e.g. `deletion-tutorial_2016-06-27T14:00:00.000Z_2016-06-27T15:00:00.000Z_2018-07-27T22:57:00.110Z` for the segment of hour 14. -Let's disable the hour 14 segment by sending the following DELETE request to the coordinator, where {SEGMENT-ID} is the full segment ID shown in the info box: +Let's disable the hour 14 segment by sending the following DELETE request to the Coordinator, where {SEGMENT-ID} is the full segment ID shown in the info box: ```bash curl -XDELETE http://localhost:8081/druid/coordinator/v1/datasources/deletion-tutorial/segments/{SEGMENT-ID} diff --git a/docs/content/tutorials/tutorial-query.md b/docs/content/tutorials/tutorial-query.md index 1cc28399dff..a5cc7e2a9dc 100644 --- a/docs/content/tutorials/tutorial-query.md +++ b/docs/content/tutorials/tutorial-query.md @@ -57,7 +57,7 @@ Druid's native query format is expressed in JSON. We have included a sample nati This query retrieves the 10 Wikipedia pages with the most page edits on 2015-09-12. -Let's submit this query to the Druid broker: +Let's submit this query to the Druid Broker: ```bash curl -X 'POST' -H 'Content-Type:application/json' -d @quickstart/tutorial/wikipedia-top-pages.json http://localhost:8082/druid/v2?pretty @@ -114,7 +114,7 @@ The SQL queries are submitted as JSON over HTTP. ### TopN query example -The tutorial package includes an example file that contains the SQL query shown above at `quickstart/tutorial/wikipedia-top-pages-sql.json`. Let's submit that query to the Druid broker: +The tutorial package includes an example file that contains the SQL query shown above at `quickstart/tutorial/wikipedia-top-pages-sql.json`. Let's submit that query to the Druid Broker: ```bash curl -X 'POST' -H 'Content-Type:application/json' -d @quickstart/tutorial/wikipedia-top-pages-sql.json http://localhost:8082/druid/v2/sql diff --git a/docs/img/druid-architecture.png b/docs/img/druid-architecture.png index cac289e410c78cf413d9ab17e9f28f06f0edbe9e..1f1906e47e1cebe7db69b53a1a456afd065170f0 100644 GIT binary patch literal 207086 zcmeFZWn5HU+de!XAd)H|N?W7?(hXiJDoA&ubaxFgj3_8d7)Xa89YgmpAQIBu3?Ks1 zIn=<+f1{#v-TFND=lA_RUljIQd&L>Yd7S53o9D`kcgacUNI)PE`TcvhRY9PW{vZ&6 z2r&`xj8u_!w`m|NMJfk5o9OpJ|H?_XnYe)!PXxVfE+gT&cG z_1UwRs>X=U+J@Su+Sb}SL$Jj z@4V^&bDbx9fMWOOx>_P~;T{(6OW)VW$!Cd(8mU?IDylz#KCOTX?*#jY;up^O56z$6 zCOR#C>Z26CCe_)mWTq3KLYmk7XQAf27*L_x>+ny#uZEPpZ~Bj3Eh=YWD^R^y8|82F z8FV$1<|@mGC?ctlonJELcHNota+xFvUZ25YB(g+-X=G$cyg?Zl24V_%*#dszf>|^f zSt%Ne#Uki0&Xy!XAEt=%?z&(4ByjEgd4h@y=M|T(-Mc17KzifE4K-c56ZLGyp6f)s zyu=PB#*+tXaptxJ?<~GLJu)_i?_jYOf<#2ypUKD|YzJ7ZcQ+Obp*qO?W3Zglyx~bQG5a$-{>S*QX=q-ttH^v-=`nK1m2}>l@t|GFJOh!+2=O1{tbe?t6UW%PeS{GV*{|19x;lFWBl`2UeR z%ULwLIbJx1Z5dUl1LAOij9#eC2i31-K3r&KK3?gT=6ATCX4RXPcIU;p+2j}8eBP^Y zkB`3Bbc>s-(JNKVM*{7uR*2j4YSsF?dOBv-wKMEX^o5(b6be&Rg#u8 z$$LTiw+PhC06kk}`#G+4O70k`p-Hu~?Xm3>Sd7Q!mlr%lv`>&48On9~etR=%MhL$& z*$}$-sf|7d>rJztsFYkcrT4#G?yYc{8xo^6@+r8$YwVw?ni{NAWUgTArxLA=@Pq3V znm!kgCnBed+L&pb+wYSNW3uaz*nB`srchcGNKRi_>%BOT`zH6DQ0qvEjRmVhjA^LY z0z1EXTU4>AgCg}`U_5D;I2^xncg!a5M=SIZ+uCNQbSwp<7{OzN+v0#AAVZ8&&t9wj1`|NV4Bi@ESa}?v>qqEFA{15&eubWvnzS)f5!uhjO&!c|F^*$1?y zN*0%W)2#cmW_kkCH{Bdpj*StQNJMkVkbG-{m+4>lK`|P1W#(i0H#KHjxL^WdlG{-# zO8vL*WX6VxuZT>e!hJD&a|T^KdeS;1mJadB`&k8HG#@lB2;0YrF6GxdyJjW&Y}~(w zD(Nw`dGNKKQd(lZMrwER#O5FZ({DSJU-y<>tx5J%D`UWCD=9|TKg>eQ2fmt|wl@-1w= zSsGj}%@>2p4YQ1V)}QKBIHwK+9$GZ!_vzUgZr@_nOKjrEp#qckZ`~>FAMU?t8>?_> zmqIU1H?jr-b!;q zSmZUGN3LOugD{5#-))_pnwtScM_?5!Q6m*_`EIDdbW@n{8#X1O)_2!kGMmCgXU`92 zOAK3BqZaz}E`E%A7ce@(8=lriWM-#5C?JNS`clxI_JFOaJ5$vy~%FH zTmmVN^Xkr!2_>!VQg_PTv5nVPGlk;eWz(gHrUqW(Ej`k7j!tGqYnZogpY0N*egOtn&EtY>~iTb4sY^Y$2;s%KP+ zA{T|?7jDn@q=sNNvMiTI0eFmJ_S>ynR=sIONSUpXJqxDeBCc|pmK9da1IHyeHC}iX zgjuOtZObh~E;RONBxz57)6CN!4w=uMTzGd~Z%n(1q%TV=zonpw$)?nP+*&h7`wVGL z)?D78i;#{|-`f2HU<>x|z`f?eyV>OZMVcnOT;(f>yHnF&x8RQvVzB2_Y?TIIPu57k8Z4(lU=J=Ir(933B)vr#TJ>eT64tSV$<^mf!oYi39##`I zz9;6A)w?9Uw_WGs05pHV?|`h<1`Q&<8O%TQ@dB-#bg-3=brPpVUQ3x)4D z2|YQMW?1?4VSQkQiDlY5E|Hk{WW!H@JvqmE!n2Et-~$@U1{`nuZ;4Ce*HTI!Q9xW(ax#cD9KJVkA9EL{mm9vr>&i%gDEc6vLU%z3G^w zkM4(S%6(H)6m7_$Es~<;Bw1R(WcuvQL_)oDyvip0szg`Hr}bHk5-*RN7;q8(OAbGo z8Z;kNBKz?DO$J<*K`SkG|W5V<_$MDq)D=DuIWaMMS(}N*z#1b&S~gWdlpoz)}Y5$@_-mt zJ_$RQF_;sVt(j}x*Q#Z5@h%OaRsW`Yu(^q9cB63f?e)>#!DZZS? z5>0B)^|mjl1$Cu%$iX!8DJ3(+;r0tuSO57E66L@b;Oqp~p8He71v@7h9Q|W8c?Dpt zs|YO9Xll>Jxnj@E^pZmu0u$3a{>g4Xeg@K$vRTvz(B{GP*@5lX6Gr1*0%t)qI@ z%u#$e+?kNh2XD@{pIEJTs8;_dHyAgV3Q*AkgBF4GJ0@Sj4OND#zV?Qaa|`|1Rn)z5 z{P7u{H!VXNvJH!uM@j$>w*mgMF*f9MsCKa&%71@B+}*ZE;Vp??Vh~Tw?xoU%Mn4RI ze#^*g;UYRp%Y?=P`-vL(;spl1=u>|!J-jA94N`nuf9YrAJ!$9|5my<2C_;84U%BigEiJt_5d;@FpLDrz>oK2$)Z!mY!5=a0g(G7UJ5Xeg~d+Q<5y) z!Ea=}z;n)Pbtjt|?NN=63fm(ocf&HqX%-|<${Il4^1^HCDQ%$ytwAE9dcBdppe$a@ zEm*C0*wDKvQ|$KC>gucDpynzr%VZIOP&7z00|6uvhb8zS7X z1^|d{som(iHD&DnAmZiYYA&t3mj(#onq0CDY3xC5Y38=QVqeOriy@G%3|F{Vv~*-L z2|E^^LVC_6-FU4A@6_edSyz=d@)&oU1#CrFNn-uBEo#uv`?^p&!C&d7AM|U9B1SM; z_SkfKZp&mk!zYnP*#`Cb2dVmyLzH=aAh~_N{di?+;r$-BkurzXcQ$caA*(St`mWt~ z@@O-`PgqIwSV!;rarb&z9q2>=*<-Z$YIUzd5SHI=M8IX#t}GhhJh8y$VlvgzW3pj2 zBbEoo5dh=70Bg!CvulqL9REPeyFH6G>fb~GY0^$-u!48T!j!n@+T_qd#;I`=6yOtO z_DhC$b0f#SkfIrQbpW}gvd?ol($~1C>wHf`ZhB`Bm?wMv@#DqkEi;iuO!5tffK0FG zhU^^^5nOW8pbu2xZeT%>|A^R?b4C8wI88vMJo^Dpa-R{NokrIzYQLd&5H|&Ubw$NL#XPaYYzbS zPV?DV&gam2i zy`H`y}yuss!Ss#5Vbg1WdP==s7KEo z+wZ?^oWOsN?B^31SrUo0+7oF&ma{VQj@!WcDEaM$8ABB9xA~4F@<2s4$gvW$0w@j|vLSLrIQU0q?`Q?Jo&bWsxbn1plW4R0J-Qs3%mB;Fk zMLLkkBQO0OoQa9wQdCcqA3J6EMSK#0P-balDzhJu(zXNeYI5@*gGT_^w*gV@gjV)< zAbJxC=knhAHU*~5KAEqo!u2d&6)0tBJ1D}nPOC|V_Xh~UUNolXYUaj}%2EKeoo`A< z8N`kBcz4}Wige?O$I)^#Ij&871Br3~avJ3GtMHhlhS+x}KUK-^GwP$)!c!)kqoLLc z{wldM)=MZ;W;^^vQ$x-^HEFxiW7`2XKOH8GtOls?iXT@*F|9v^xXrjlqVIqpahI#% zIl|1r9G&Tvovry^gL?dBBgqhW-xs8JD!~r^e2hmEqld^KrPOXM;oMP5mXa_kXx-;H zYClqZKfB`63Eu?vzd0%K?NePRaB0yQ`~?*>kdai>S|5>x!LO8+hAmbxUhINFQSgKg+#CUToOYNEw$$rJt_uX< z2nH=S2I=@;I1SgdB*Xeg_252m^X0H8ptw{%ynvT$xBzshb4l$|Z1_0J z1N)nB8kCZrD8S0HcMXKJUjBr(29?RVM2HYjn&Jf<&}057O;X=^INRa2Tc4_*T18G1SP@^l5SKYTDbf#%j=UL>+ zE5B6QCjI`#)47iL#U7zW$2c1P3%ZnbPY!#@g$|CEdHP19hcR)-^?K z^=p??|COViZe|7!arUNL{_bVC#a$>qEqj!H5TTzA!|A_FejEUQP!)DRd;kesPyCldats^E2-dV9D zCbGEN;QXfmk~~kH25}qi#Ae_5r3vh}(HLj9-+QpS5H<|N9JM`BCzeHi00Fc$T zs&xG`zg8hbNBUKO+*LF3d!=mDzUq2*|KDC^37^Hk+Iw&y>Wi~$KDi`7kzRh1M@8|& zeN^k_Vc{*D_G7{fO=gp!oJ{sm61n`V75eB}_@Gr1%>ko?XFkVKMO|~I=eL>udiBd? z;MFfKunN`RuL_*b07~DYK=!ct^}#93?5WG*f2n)+$N`Gf9p_d&_sda|M2NrMp|gC9 zXqbBQQSi68e<^<*C@qHyKcx629}xai&hSxzE&>QCYTL>2*BX8gsZ96{Pe0>;f#KI9 z5Y!Vo_5li%#28I5jc@o){%$$`iCyT(Guv}#|M|V;p8G86R!Qj(q8}l>=Hgu|)jMd8 zyW=1L3nk$_y!fD~C6eEb(CV|z3UuFkQsrX-V2X|-dbPPgH$;iEiDM&?sUz(=Q9ju& zdI_gdhvX|O`4&(ER7jNaZ41A6t5AZ1>z7E5Uj@v`^xcvDR3_^GiunFAM5jLV1Zcs1 zQ%G)3veV4s_}#JDMe_iVvblOg>)9`h3{S%!uksKEs*U$Ow~ve`5rV16)UR;n7>~PR6E$inY9?6AWf?{BV~4 zw`6}C6PpaEe0SU>@BA;}v4ju+9?j@cYPdEikM$XP7<7gpRP7PbuUURAMOlJSZmBTU z(@;$7H#qvsbkk}Bm9h#=(J)3UwzJ>M9?fmPBQLdeWNN|Rr<=O@8`*^wu-XT2!uUl@AlzKm5; z>syZL#sgH36q}ed4{=-u+uBe4j?HcnD1K=6`O}hFe2*-vW+#;Y?r-p7#RtxGobe`e z9wYoHdTycxqkbh>$5I7iN*JJ<|M1aAtTLy|qsAS%X@80O*QojUFY(%%e*FrZPms~9 z_1pv)zUT%C+_mdgAlY;Mjt74(5d__$0&X$*?Y!j2Sq6(4-s4&hwE|q5{%N)0XO3X< zi;(jf95cRhj)gdGDDW!IqvI4I4Z!b?V(}Ifj`^RyD05l_|EbPjx)YHA-D_i8uHu+I zPysuQL-l?cFc8$tk4Dcyf9}T_JzU+N#Zi^%++V=R>d0}H9WK)1jKX>Lz)n5QII0{hPCHGq*j5v}ekptFq^S%oM&a1XDJZ^SADh6u#*dDVwK!P&+q^AiKvEI!zI5Ifi9}u35 zl^!3TQ5}I^Z~^G%rW973Xo9oVfwv&$-+F1wg22fc|%86djpx z6iJN-n9iL^R!0(?hsVai4QzJ~KTH8f)Xlen{?VRV6|6Xu{typ4_YtJG#?Ol3_74Qs zZ);q@`4>2OawN_7v2~d^l2CpNsEanWs;&a(;jwr!0rOF8e_rz2@lLb9!zX~gn6aBn z#+md-v|tH2;-Ty9(|tHi?8LzO3E#Kfz!~FbaDi4IT@VruUBanXLth@bN_Sc0i(AJ! zJ$?~Nb{6PXYsK1^z$0+x09+-~-}Wy&@vADJDsH@C97=;n@E90AfT{WqL%<^-IqH7X+T|LK z7=oW4Y1TQV;!jnvJ0Cg9*!8I=$F>ASh-R`Q$8ut>Ai){rrzP+Jnw@t;i0I)o0pyP5 zKgiv&q4CK8%OM)O&5d(y51M=p2uJJ_j2+~Egd;u$pnux9N*)eRgI;0>`V%9m6;J)y z|7g3iNKI}WaUL<;)8*gBsXB=N?5_m$pB-1r!)bzE;sW|NBk2^${_KA{=@Yje-&fso zGMo*czDEUJpKgi;PD!JHO7@)uCxBnH1Lu_wfCJ4-k^EH92WNMe%O={Mow>|6|GNC# z&v%<0@dTG_fom**Mpq+)02}%M11xz9?h%KZidbC-j3L*7nJqC0pNdUUV0pBxvtF{0 z>v#by5JD!%fs;$WhStpL%^~q4CzA<1;tM8XLb&b)9U~5>5EBO5{^l|Mz~R3wa7eL6 z(edg#6cthf$Z_?|zbK}R9}q`6cyzX7x$%KIWZ?xVAJ57-?zvw;E&>JazFhjA@+m(d zl%t(;neM}}ADN`{#GW@NZ=2v}d)x$QY#f0JK5YQy?Hg6MQx(VZAKtoAW&NL21VF2A zcyWo9OZ2!pP(8c$xHf`ID^!1i6#uBs=PddcDv3vc0dg8+sZ+S#tbB!_nuS*q`r^~_ z3*Pe<0`!<3@Wu6<9QpfjB zJ^0tQ%1K`W1+^1Lp7SHt0(*B1tjfT~$Dbr*1jKo~fsYkT?PE9cKw zS?mabOE#v!rHP{QO)PLytk4w7I6ce%{ibG;<>gR$3Qm0a8G1|rys=oyo%x8fm3t0A_jzBeh&S#a-(O&7 z0ET}$-Jyaz5(^>FpM#}<1Q*bM;N2t@3uvj{_$==7gA)M2`FDzW+{|8i1UM`3sWzPD z$V@5$#VDI5e~{Yl&* zeqy~d(EGEmcJOaF2jSm&z=C*OJ6dYT4^)^qA8EDD;{f|{_#Xy#9Vf{%fE!z(msr9F zZyFOqyn(8OAy9|D(-Ul;gt(nLx}>r>*U9rCdn*4g`&|KSVGbECiMu12izHtK zh#4p_;7A#s|Db$7X1RA4Sn#~(poj(Tf{2Mtg^AkVUiE^e`{r{P@8t5+2fc5W@KZ8mjt`=?|fOI12f$ z6>4#8xJ{-UNi1u3nSD)xlTC(-yvMb^=xv4b2PmOr+Wg;lY<_R{L+sHgRVHyfxT9R5 z0i2DE1b=m;glm3V_Z~<@{*vaeT^tJC7XMSYhx8hxc0n}pu>7@3%^6|*~}lhQl?@Vk6;JOOnfu*-jH@EHDejJ|Q#7_jLyk5wda#qfrx7|1Hg@N^^3mo9V2Aqa1vhZjNKFH+28?)6 z(Dj?L!)x5c<);dhnn{6z;6~Xqs(%GZbpj%?^TtvKYxU}y?E_*CaFk~0fQ8u~`;$rH zJyJ=URMWqlUV>_ObOK`z+zEPU=!??m&D9fDE43Y#7#6ll+I;dV{5PDGeDbx$DTpBkp5jc`O#_ql>^a+GFP2|B66`m&4i5OE`aRURnxt!!^jC zVFjq%8%*B)>i@45$k3hgtTPUwuXdU15D;@w{Ild2oC1>ce=%(^7lS({%U%4d@*7l~ zcO7k?96RB!4G^-*EZ>2uxUl?bQ2((PfY%wyU)N_ypMUx2YhnQNCCl? z$a|F(q-s<{6*iI4HIKK7GOfO+Sm8|g;_k9o^Z1%aLBlT7p~hYUT@v z?R|?a-#}}$FWjSavXc=>e5gQYgCEkO7KHLs3J8Jpvohi}brU+>>Cub=Zc}^Ut0P@!a%ExG7xII0zCeEM<9^{ z{XLQUW89m7&)lR$;^OZgeZEg<1z5U>odYA=U(&Z)QrTtgHdI zN3>uNf6?WdhXp$is9)B;LuI#bO)14@I!K!d!E`Qob?Ks+AL;H?H!?WtE2(8|F}+&k zrTT^1$G#q%CsPP!#Og%CRO|aJhsKurGoGTe$8+L`Q+x=qoDF{9#YVT;s#l${9JkO_ zjaw=#8#cX*bS}nC{0j4YW+|FDbcTEV9i*RWnOduv8> zu?3yuk18h)-(`+3b**t$|E@m&Bt+0l*TQoQn;;`uXr2A~Cqp@M-h+bid5aEo%mac# zJP5W%gfr~uCeaeFTTB^$$cV>tI8WR2M1;?%&(@^bEG0dGqNZOgHel}?RK7iG&R%eQ z#dEyR#_d62zs~Oy#&B}L`G#v-+Ou9D7)^xobnQ9SVG)kPqe2b>q^y2%j$UHdv=7F$ z`KS8P_~|31J0(^ZKndb2k~_Cd2Rj&{i}$z9>mB6V#f1~iJHQ+AYIYN8pEb>Xegc5w zd#3)i`w=hN#mnOCiyou+nv77-hKqFDeGplvPGIfaSB99x6&Lw2hF9=wN$t}8DObch zmxi|QE{zq&D3_K@m7xSIP|V9x~(%LEhLw zjQP9vI8#5PHIvCt0+6AQ@wPo41*O(nkI6@#-`Rt54gqGUx-MsbhHcbxS*ytGkPhvu zH7;a0+SF&+H|Cd4bNJv&v~!XA!%FvIpPe$VacecJRbPeF7`J5`12yX~zG&xY{gXeC z;{qG+;){#s?LA&;t7=BkXRpeekD<)mZdy~rWb8$D^=euS)0~3d=o|ZNR!+2&#NM%M zh;ukmrRMNZD{^DQRju@&nOZ(10LFEs88w?hVMfBzBZv?L57#c$aHxgW1<(*6pIvBAv7rdIHq0AyS%ak=U z(n^A8CRc4^=IS53CyAY;tt_M~jU3W=Ep9_+65S>~*fX;&Ku)=AJF->X(kHol&p`<0 zbuB5l>v;fEaR^b+u&ia3nix2ULno_vS*NcT8D#Sy_k^x@k=M^6J=BM~D_>J%aiwFe z#INB}N?!v>=s}IK`Pi$-&X`AOVV*)(he~6z7x-?$A!WlvPcc2a`EL};#3!EFu)y3V zDoTQeYs}PaQRIm01(_vHv9imGFWXIk%$`J1W|#}Ie{g-XK0~X$rDB3+3@Uo|NbE8- zM+BI5;9ER*^m>U7!uk1P!x=*pG_ziN^(p2+i_MyF0)cl<$8irt7Nu?ZJ{S`gG6LL(*K}btVxZEMpEOQ5128oJ^X47V{rxb?Qt;K^40444tW3D zYZ{#MLf0M`7&mR(acg|(ObNO=e$R@@!f!tf-FqG~z5?!DQYrQ6vC}?2AFQD2@>r?4 zP8l|KKe@mjLFZUCjE(9wV2x-d=XagGBz~Y%&fq)jd$cq)1j=bPLw-fjXbT~g|G9^` z<|2sU^b|S%9#_8IqB+0A7&1PX5g=M0TM&Jmz#q zc$q|eM35(A-!n1Vfejfw`i#x|-g1@%W#(%! zs-?Q$a+zF*2%;H%b(0EJI7redAp^_K!(kJjmiHI&JceAKnvMIQbcg3^OPv;TbVXX* z_P`%>Zg|Hmi;9mkd1s>Es~621Jt(5;;8O_uW-s@a{Y+UrlWHT}Q_C1*RB7}mYi0D4 ztuFlIg9{tw^Tsv6mWge0|`k}5J^M9y=^UC{QB1Ia)OJ|tU-yk6k_udk+V0f(Zi+qe)Bt2k6vUAziU3} zQ8fhDa`-yGDq*L_;cL-^KVlvVKfoL06D=)_7b)(PZc7k4R4va)@7CfOV>*8;!!LIF z62vn%ZU-K15tizm>dm@HB4{1!YwPe~HXhB&;9*(B$Zz^(k&=L7JZ-Q)E&0$V)9KUK zGA2P%R^LYhJVurj;;Wi2-=c*-4U8B1m8dlAc9y0eh*kN#v5UwEc4XAbyb;SZJT9>4 z`i{dWRo0gN$JYpFNizmpym#VE{EDz?>H6-=d{l<;&TEB|LY2tv;7t8&zn0j-N5!?= zHv8R&k>#u9-vpEAvIe(L?!?(tPDqw(r$xQge8~L+v)n5LnBLt?MF}Zw1DS?adVE~A(EI3fR>L4B!4Ch-%4_hs|wP^IL(U<5u7Oy`vnC64|>>xo} zb1Cu5V-=V4I~yi^XXv)A%!h*`3{v2=B-a*ParZiL@)15Tyc%|dN1%TqPiR|z| z`G}WdrxwQ4X;Z{9X4$sf603$}ADCAA`vqjAyzUXt_ESxKqu#o7!=b5=HPmaLbu;^L z=j$i>L_xFdoVN?y+LYft*#9U+z3jS|#DfEklOb^5h?4IGDjgFn3 zbtrQwwb!?4B8Z!R+-4-9G9KrIZPsS-fHO{;ZRlY`Mmu6}EXSu;-&cdzK?{}z)F>I$ zmID@s=CC~W2PUa11Bt38T!+q)emtK-q|58K^+}kk)*rIzR7K3j>nRHB+17S*VXPbs zlLr@2l=;)!ZR7Bvgt^G=_p{~M82Vvn4Xg98L(iQs_&9x|Qm*b+(canwZH&?KZXG6> z`5DOvPJ_O__lND7J7hQ9u+DEGG{T3iZ#&z|gyP?7^;wrbB|i^w#Wv*K4Hq_ofio!4 z-v-VYuz=ZT|I;lP5OmODNL}R42h-lF;qDS}UXk!6n@>S6#*uRU3*W_WtRe zuf&vkXs;zkNiWT@`$&$59%L|GwYFapTf&bP5bq=MkRQHL_HKkoHIYtmWiwLL2(d`#!00v)#|$5TP9%dz zlHoVQGtMLU2InT5M$CfI+;%=5Y!r&D?hdwN%wZ~!uy8jb5T_Go-IW8 zTgx?QKA{^ZhR6&bE7u@{fI9y~UO*+jP*86yZMT8AG{uYM zAVgiOi*=@`?e=uS1>I^?wV1FkYonA$?7U}t%8JX<^*KR)LccOpy|k~n zngm3k%dMvQvqs?JY~S$x`Q(0HH<;{o=R*0?@3g0x4f=Yo*vK87E>olkA=hSaPCMs? zNe;qPlt?sRXM2)Q2QrIP42?s+(9-?4w)GT@$gS37_BW({?$~2#cD|^XZCnMla4f)cZX*J6LmAd*dsc{4+V1nIg$xvnt0c zrh{;Wf8#MpD5<|Ye9cOw&Ik)e7C#nXmfjLxwF@=u@A6qIiN6$pP!aFzV!pg8!Sygn z)SBMGF#a54_2;4NK$vH##Vy)uje)`pi>{%TQ>=aL2N9&Bw}~K!v6d67L;iDHQ=PjO zY;jCGJqAbIN`Gha@)t07=N_lF#g}Du@{NN#a2EQK#j~-gl`qR|$0O--f&vf)5tumT zfc=Yu>6$ww9or1dX_Y)iYl8?xYH8$Fer)B{9Gi9E%j#`b>=r^R+F`;MDIGgcc1F1o z3jEw52=d#jMwAngwp2%sC(BYziOD)J16idI%*gM_qRLYH^!I{T(i-98@l6t}58nzI zb+blon~`}46Wu+#D$D(-@{>X(`?1^GD+xuI<-=YhiFL}zgvQu(^;X7?c?-H?$vV_V zKW)CQ!@l0VEVBQw9b=xL{v%?bbEsmsDbIR=3QA4*( zD1|f@JH~OXFqf28fqpqY)Z1=wu2FmOPO9ui%M&Q}#zM(FPf2W<@P1v=$yp;iCGFIb z?O@ddpM%K>03^OnPjiHxi*TR8)gb#$=}1wOE#=sL5W`}} zs7(aDn{w*R-fms%iJfd8fC?D82?*MfR)mz!q%~5GnD*Ldvqy<8NYhpzC%uKOE|ka% zD{(ye7>aCJh)5Nd)s)18Ao-9vqlbm< zOg$T4H>uJ_Hes!<)&6vE=KXeCYp#doOWJp=bvpN^E_>Le+n{3s+Ou3(=*g8+1b?{o z^1bV;F!#q(}KNuH*2*-Q%cx)CsaNg*YwDcNm> z&~N9mPxNk_f?-+*K8JLh=9E{$0x0;+TC|F91t_@3lT|;>f}p>?=W5H`VRiVr(@%=E zSad(@@n9E;RP!Z*t0E}~2lkkiV#T)0E9c}+c3yg4ijU%Pu?v|2mYf>82BzFk+w!mhL=eKSftsAP-yAO*4No*9#gVb&OPhoj6k z^;cOj{cZom$_|l&9P&BhA{LsAMz$*lh=eKKinbiHgIrL=z7wG|-y-&?WQVEiN5?9d z+>v(n?$4pvU}6xK-~M21!cJ`dz5`M3Wkv`J(ta`BXbSkt1%3`(8G_siTC<_vooS>A zR@Otz_n!wdebEp&7wp&9S9sGR=xvFBfUNQQwxyFz_6bXTU|GxT7W9$&0HSooB_caIM6_me;cO!D~m%V}V3Dpj<_=9@+8&_yCN1th8FfaqEHQ8L*5Du;91m)=q29X8kNo`!$w4j0n|?v1rpHK z;$yXOFr*+Iy?up(OGa=FP!w{CfMi7}r<$lgjY!a{DTF5+rp|n(EnjU_D1A8A_g*Qg zVp_J+sXJz5edJ@Edg{;%x_39xy+uPi%tR*^P7bZ|ONyIm9F^Tv!bl+;k*0CgKzWCb zyfdlM3$wAP^k!+-E`8UcAuZy<9r8K>MP%@B?-Kt6J~~ddmz*?tj)=TSrA5gWh55`Gg6LBWT&d;g45i>xZnyWU|h347Uq6@ z?7Tb+{K?%F##nY>Tg*0Pyb4r~fDtAJ9&OJD|QhINv zc-aW%vfA%x-xA&w|D29I;c&@w@LRW@P9}q!A8ESp!=}%A4~0e%lD33)PwCL*oi90S z=|(KhKyHaT=>7P1BN*o2!);@E1)@@^!2`-c!lsf7%z>cE5SuG&Y(0e9o5rF$dhq)% zXDDX6Z@RnZ3`iBYv5_5kB(W9HS$Mng7ep+5lF$qL!egzTJF$t}K`iiu?u{Z0>YX9u z5ep%Odep>P+fj9;hL=^&@kpXGpa@Kj*P)-5!+YuxJFt;S5KAVbM>9}TT?lO5IHwMS zB+WZvpUIr~blDCbz}G)7+Q&s@C%4k8SF;Qc@z&_Hge`fvQlfJ_K4jadDr?lK5;h;= z8ON82aR{uA4UV>@85^DxNjvvVw2KR@NOtm2+;B5WLGn=8fo4ano+w7zZ>?}h0>wFC zksb&ixZRK~e`?UztMm1XTW%aAmfk3t%DF5aE6adyopI|3A`-FCz~`^tYcWTC8*1aJ z#BW`4bdmNuh*8*-5E*#RR%M~rF>oSVQ902K0`oi6bMM%kD8D$eHF`-!h3HOd!K(Wl zeV)BgH@NM4jkp;C6m0h#mUQLzn3GXgGf5t3Xdl8JaYb>sEi$eTQ zPwm`l7~FplVL3Gkb9V_Rw}fD)xS;BH-XMKGg>J@3Tn|yDO$2x9rRk?ybHpiZab|s6 zMsE$4KB`L85fNNumZGcIk~U1E@cy{vY5XD6#(T~4&P|)tjrGV5tdG#?gMPmF)dHdZ zl%8GB88&hItyi?;jOgxf?K_Apw=A&9-H`6yjQaa4{FAFRxvrha0K_(xeSvNr=9?jcG&0Jt!Uz|Oas`?zT7=m>TC7}RO=|X#_7Xo7EIpwmc3=sk zl4eFiU0f(l0cY-xNoVGkTq^G?J!jxxuwLf5J-dE%7EbMfNBE#Wn?-qv|DVMK2xXhx zR+`w3BAY?v82I>VU#&77@KHOzTA-9ULe(9To`KJ; z0`d&>RgB&kNU7p!_cAO&`@YV!MorUmUXR+yjJAWBNz*?ITADg;U-@kji0yQF;HZ<;~hxwU|^UC zGqvs|Zm{FYvYUOl;B`x9iO9f9eTjDgl>y433t{85*7x`l(s_5l0XXf*;cz&S8atTt z(k`!(8UoUW7aiUSB)5~5g1sz*Lk{O_G&!UyQiD zxbu)smV1o&YCpd^{L%C_+i<4`bMKJQH3juSJ+s8X**1*P$VKP}+~W5>!I=fnPw9bDkumvZR2>wv$c zf55z^A&hC_AT06Sl~;fJPRD2qlaRM?asQ*v$*9I4w-RaUP?zg9)TU|6b)!2<;qfCw2@TJK zb@A>JLP}~>IRr~2UCjL|t2_p(*O|-Z6n&BEAE&X_Mo7Nvlx0h5wNl-O)~3wo;h_A| z{zJKQpaBb|9r?_#k9VxHcIvrvGr3g?25~m$0^=JTQ7h^4F+QKg$m2Xz>-(Px6Q*YtW4y} zN1WHdW6+-O6wb;~<9VkcRJ%kIQniz(kXstb$n+1~1$8ZrI#5M27H;8bQ(iD7^0Q*>0GdK&mSW&Zq-)gMSQz4 zinBRvvQ2{DHV3ADWR0GuY&TO!QF&w72a}Wx;8q}mfA(Q@#)0UngUs>-YqphOv+63+ z^iH&7U$}SoR`+Mk)&V|A+J)hNh7HY!WtUqflZh-p2h-_=l;@_mzMVD8QQM}3sLTqQ z4Zp|=Q#OGCC52$}?9^qy(hEBj@@GNz;ICG%>Wu8NA24UpuM6wy87JD6VhaS{eG%1< zOqR7WOu4sQpJ=GF8qt1jVQjpRqI9K|7Nlt)cVv6$g`7&5@|qU9$0b7bF*+z<)Y6*^ z;p4~UL6Ie|vRw|hi;rA5V4r78nqb}c!CIB5}a ztpNp;2HbCJa6RU32w2FA}y%W2R@nT?9{h3B%#r=f1=i&;Tv;fm0?S)t<8X=1FwdN_npieaHr>R zFS=N5NakBdO5~c50>$cVk4NB~_v&b(Z;{vPN)1$&%sMCO?Y(_9l`NYO#=fu5)W!m` znPfS>AJnenXJ5jvMKcVs`ZZi}8S6Ut{;iR{XW*3xoBQsOGs*Xbe2sv>+o);{QtzNx zO_G(PyY!JuPFM)Y(PI-zFKQNUpyWNSlJS;`tjEdls!fQC$(Da{f13NY?|d~T*{(>9 zV&3WL-con=v{3hEAg0D|T~T~MpImKldo2UT9B_iYQ-Wl4ll?1iEfP0-AR7;zMgO;i^!y{?+qQYollOHE& zQ6xDOqMq11+@0CUSIyI&%yK#MdNa245~b&l0Au%>_H~};^*KXVRPO_AQy3X+GNBFMczYY-vfVjmK2F%?-0xtVS~I$nzdF(Rx)m)b>55P7pVVJ^X{#NZ ziHf|mE3MfE^`e$^^4HbDYLZD%0>0i_7(1V;>PEVYD|st=G`znHyFR0`7oD5j*vSQ$ z!%UgX#K73HpMh2I_&E!HvOf-`&`{280-;=6;!pL>;cf;{xa z@ZNTLPF;s#UGQs2ADU+GrlvQN`8j3(79AuadrMW|`Fe)TCHhQQ7$-Ty!11o)K{eo0 z++vJ48L4X5Q+sMbWht9u>uc%bJvJ$Uq3^x9H1ZJw`B2wNeF>M{3Cssby>x}HkIhR4 z$Kz{`OEw0%Zn8GS7;Iny5`Cl5^4inF#127Wk=?UE|NJSWf~{#+Di($tQRkwJ(jqPh zj@>U}ygp~9aevStHbW2XoUWU>;;A#kt8pW~LL#>75inn61C{fY*+hOL1G2@7vQ=tW zuFZ&TR>2w*nr2_!IJ=9!azEB)oMKQOBWZJzlhPpv`1rjQ-(tW($@q=8sCp^Ytk!!+ zu)h_1fTn85>naK?@12j%@kAkF(UX->u@q~dP<0vnNnX7P^%%a`+L#+)Ekbm|FGB50 z*|JJDo0$3hv1Yd6BBu$ySjy&7Yq_+BqPs?r5^wMzgMdjn}J@Vm)Uqh|3+}<&qs~#sVG_cvgLbC+ZUUOlWWkEeu!@8Rt%*tMRjdseb7&itE2_wd%C)5#D>Ub7r*aAPXn# zZLC9uyvN@?mKIt6E#a!*yh}ZFZtktJ#vFV_nIr1jq`GAp{>VprAa1lrTW+sOCjlr7 z_4J_J5QbwF^8wk85xTtjnDdn3nMzD}Ys!GoPxwYdWN4h46tmUN#UQL9qf=ZHp#9O# z2kG5o2&4Z}=BpXbsROR}X0&~@IIKJqGo*50(vlH5gF4jO;0G7^-~RUCj>=MVL*Nx{ zlMv`|P1DL`w8pvUr&bxr9W1kN-3e~R?m&Nau<6t5B=yBrFAUnMGa(XveSD=dB_}6- z8=ty|*UeAZr;o8x7k2)P^C)kv0|XIA#p43^Kvmg&T5%Hy!(dtft;VCuE)A&8-)E*K+>Xj||_+}@#P zEV*e0ygC@EZxH*20av(DKG&a8W#!}%dNZ``nM{|2U8Z&3vqpidvE+jLauxS7kNEA1 z>Gy7K#|t$UQK>)j?i&*;OJFLm&1GVqtL%$>2h1_tv!bYRVz{;!mHROxGppKvqtY#5 z8)-hBFDd-(Bt&auk_ls%D;uSyZsOIyfP2q~1NbvK?Ma=FD{qWweHOd0;+|xq!Zq`h zcvgLV!fzj!%2rjjPlu5AY-v~Se~jqWZ2bDwk0xJh_#JN5Vqir|%(}xrZ)w1r(7VBQ z;ocX9`*~x%O7C_3_r4o%yhWpV;dM4V>yGdu=YgrWfsiRf${%CYY9OQHdn}&YILck* z^QQ6ACxXe|8$;bz^cHQ04)Z&uYrA$CH1EmXyr0E>I{muw%0jcpVfeFq<-X&ZHx|Qn zoqA1{@`6;&fj#qm}6nrdb$ zyb&T@O85By8UnJS1>xKBq#i_Mjoi9)(j|BQaQPgUeJYbWKn*(n?R5@ic;AZS2uR=-^AQ)!`aNuCW|@8-dr0d|_kdlS36(A-)XU*t8`^^ibivg{5X-1N4Yq;)?1iX5kFcr+u$ z)is)HSe$?3fHWYMQ~z4#X|D_Tjnqjw(zd`h;eERDErXdCAGsBO z@2H9n?7XTv$m<^=FK&4CxG=9j4w@@Ibu{19>eG0yHzlcnQ4`DK zgHsR{S1y-bYuJxJ6R}DU%wN*Q@#i!2Tl>`Ily2vZIsEX}=unkGG#(!>Amq+8*CBIe ziBPRB%DYGt7-C)ySqSpT0niRWZd3NNk zE%lyZI5`opUQJpa1xv%b}QhM+raB`{KUPz-t3x7-x@7eyE;j%6kkrE8s9Awne0n3d^50V~B0r@wl zvy`~_k&O5@XI_U_RQqWI_si{>;=Qq)Fjw{L5Ot4yyXWK>A$*@t-}t9qA+J)Uq!g`$ zDJUZ0U>!}}{bLSQy+RAS_Y8iyZWq{Mk-p1o^)pUoaU?Xb7FvxoGQ9?dvH`!0Z=+*2 zYYOF4vz;|spUp+wd*<-9%(HVsNMm^Yri{0=*4_9}@1q}SGXC;|I|DrU!5&C|Kd_)^ zA5qD{?7DrrmP9*t(8!vB`kKaB%qUInM|Fv}JNZw;htn$}8LYIrl+^L9q&KTyG(*+XNqW2W z%CQuvv0+mAbU)j0yPU_DWiv0j-)iGmK`FV38kYr`y%KcNjJKro+HYRwCt_#;cZ#z8cER2EDVJH5_qiG$SxQ_8Yd2@AZSn+v3rYqc&Fbs8wU zg*jHt7Bc(IydIR$5j~7)pF3VI|MJj(>5aGhYQl3$;)YJ5-oTM=GMGl%b4;E26gT0T za%f0EnU>G?W~?)Ol;&`BL%rB;rbNi$^*zd4Uz|nSp0znHWITm6912$pUzKO;>KMZQ=PD-zI63|9)+%utErtwSsgDWf7%C2!yd(% z_?1KTiKST8%)WL>+m*uH=RnM;)n=uF_}RCC*u89HM>|L6hQ-K$l}E7iJLed8HO!v- zH)?uB%FaKvUSD9MlaJq_Qt7nL|X z_>_ks5%KHJtgmN``FK^2hVj9xl8$vEMzjO2lNm>Aayv92{Py|KKsSIeH+Ly1&K0?B zi5lC#$Ou`r0w>TNG#)S+TOn%?l9^K&=dq={ylIJ0mePdlNZKN*VnCApt-yMrpe_XM z$?p4c->u^Kxgy+JVGCb_zZxpCF~YpuZ3bjN1A+7mCbo*}uPjq~@3?~oL@4dn6`_e3 z;_fC-PsDCy%68MPL^Urzj&W!N(AdUn&*q$nxmypl9GdDjbu3SKf?Na#&l%i2^nBp@< zS&}HDfgxqI*D?K%roz!qJgkQS?94G`^(B0NI9?I9s{&$~%cS$LX8aSh zD-sv;jj%1TYsbCJHCj_T3YwnfA8I(lXZH55Yco$8(0%3^Fi1g~<;k^aqKU9y_Kw;ZxH*lf9KbCaIM5|pM&yoAjj z&!dFx+Zil1Fe97*##$i&hg6%9&Xt44Vc`QBt`6Cs zFb3~F5b+1F&M}l}C0#Sw^Q6OV>kMH!X9mZ|9%Kf^Wmh&kw8FVns|#T)nXq>yG<^<; zCY!Y5?J1^p1ynk~JO>uEDWpL8Xs=N=6AO#P%$O$&E-FF<>Z_F3Ji34Y*x10ri6J6P z&T%@kT!427UIcTud$?+&iNohkO2JaVnycri5bd`uU7Gaq5HVLkj=wM=t#RC}uT zF2rwneX;*@RT50cJ}$ABH#Q?{`}h>TnsM-i`4IZ9YD`2=Q2LvFfA5yfCILg@#g+5d zaw`4nmFpIQqb%+f7*1*(q_T(Cr*c_I?51U%L#~gRUD_?shdi6Tjhn!10CM7J*aS=M z8S@+x!|g8Z>9DG!s<@YDK~DLP2YfC}*T9y~Bo97yzzAX+Os&7H0Y1zO?c#y*KPFhF zpYTwJRL9=alZ$|Odznh%aqA_Qya|~FQo68 zRNw)nzE+k?c=I!vV#b>Q%U8LP*Th;O`g?U{e(UOxwIr_*DMaYU1x{>Xby9a+@SSv? zMmrA8)dz)^9_*_q(xg%g{`kW6ojSN$yXCVG8(S;IkENINS>hY^!%$X>uMlNFVIEjE z$UPxlR`Z;_HY=zIIxH^YlHhsk;lb1wJD)H|77XPw&ovcTCK}KbGd;!g8CHUu+D5@T_*%$nr+t@!hR@)wNr6$;Uw^)j>=0 z&Zwm4v|}u(1P`O3D|FG~ZWh}0y>C{2suS z#L5`%rWIQWG_exM;o3<#fr94-;n0R!dy7mySc@e&*T#6hxleSh9^c|rD5jg&L%4^U zQy6zEfzqk92Tc*n%RZK6g9uFzs9szfW|`0Ug)xBM&!Ey-kA1IrVh(F{uBSHA_ja5j z)LtmTB*-eOgr|7sK9Jsiy20znQ~ z+cDRH5cQ;5U4g7OYFt82^RMqJvg$PI00DGM!-}JG2CvISr|Fp**MSoki1fhmYU`dF zO(S*g4&?%aLYxl^EpuOztd3<=`f5&qo!UkRKFlu=mjk&wB9llS6gKfFsuTl}`-t`t zs;P@ijyB&(QT_#>&B&9gou(4q9*H8q_WFBzgFJ$nGmkC_ig%BQku{~Rw{Sljk`soo zP^Y|TthHy|Yhv=qt(WWW+V^#gB~QG{pAU6tuUhksJr>{RF`|oOTbv1Fke@MAatgdA zK5>6bA=jKyOt3n4aEX8J0-i>*Pfn7JzmN2o zxn<2MEunUsiPC1wD^S!9TL((KBv9gdge`UP=C2;G(b`A-hs1HK7`T_460xpW_Oats;^S4~?AE^iCWVn=if%3_cFG zd%*Z+(bBinLf~f;Qy|ySYz=?LLC^-4Qd@`;8(|Za=uxP*(pH+#v|%v*_?!k{bnpxo zyitjiIc=iclk?HvwBn#}C_2Tmqb&D<n$??b#oQtM> zjn(u_saW>S)V?g750!kPLtic>`)F;om8|IG;Old-Q*qi;-JS#H2)IMk+Tbv`U@P3IfllN@x zWTOhU`bR^T)To@WB8k?{ABU(gk`};^vdJs;7B9cZEE&r1+i`bWmEYw<)KN8{aut_m z25nAs1NtR5#Rc^Q4{bqH&2Vavpe9CASjiT0b7`$NSE3AkP@H^e9n@k}Zw(k$K#@W{ zN{F0tSW1k#WDT`XDqKT+zmH{7R!TZ0z&cBbTm%wlVk#4q!rePPER%d&#Be>lfzQ9` zr23{D=_aNO2|qEIdW{S-JXRvHMVh}}dcX*eCrWC$k>@2#6r$_Msyo<#^B+K?U#*^r zXL3dEF$nlcIyq+AR|V>U(ASk%5r@s{o~_)L#{gV2P^7A14L*^PU6&D6-2Qx?xG2NJAgg9Tw~UX3PRc1Ao64b@dhKoZ0U+Jt&ewW`DAu% znN!DAsFXtr;Tz$82j74@tEPK(eXvXUX3w3jn`2r2V9igtX8hH;>{M~P3NnNIH;8t} z+RR-E3FF5g6dAqG@LXb=%tqr3zy8tEQxtGxtv3+nF_`Dl=*EZj1bp{;L5zK5pZne4 zd)d;f-q8U3$T@-f?10^7wRv#fNYe%n-zIq31$M4Hv(F-EM}2dmgCkq#%6%{2k2ioV zitAsaA`yG_X|Rkc`mI0gL-LjLxC?i!Fxl}$#7QlewL>a%{c5V=R}<_xKt)@+8)^RC zmy!nFb+Nk7F|6EEcP#&skV)Z|wCA2i=C;OG1U*kR(W_ZFH|F%C7*f;s=0&>Igh`8V z(i+3(PG0p?OJ`U}IXLke?dC<8ou?OSv-eKC+cE`Ha~cAcDf>BhJVz}uASk$wIY+lOcd7gLVg0F+W zUqs{V^IG&;uITBrLEw!Pf|Pg}H2WQl+xve+S!l2D^sPL=yZ*{MEeuxROwo1R)E%C0 zmnGL6^9|2r#92~oRq@3Jj-qw*%A2dqptr%5+h((-kO`3cXxy>TbB_W8bagnkwO7m2 zwn@<&Dz*2u-oTPMY4;Io1CTOvXOHZ-z|d~is{X@G*0I@qIo-N8GSOfg4}NQmtTLI! zc8SiULwV8%q$LPgPkb#8db~|ZU*z7UU+u`;mmt=;=Qi$UOuo@ftd~~Q7;5>dv?2j1 zbdo;geu`#03el=Q0M{F|RmJA=r#AYU&mV>6qA;-Dq?=hy#yKsZvEIIS4-kZ2`y+*B zKU{piA&Mi+y#WcR#iWZW30-4!V=B_&RA=;Ib{J}~SSSOMFG+3Hj#gJQbBSM@MDKLf zVrJ^&_LDTlRnA!+)L3c{nV$uuS|%nQtwb)0Eho)btefS`5uAZ8Te0g+CJwFK_fT`5 zWF@x9yl}*_p#QBiDDcfHzg*}SNrepxgUzw)R#84V0r{j45MUtq;_*hRl1H4=Qao%S zO|o9w2Of@@bQmQ?JtJ`aD}! zoh5b`K2lFR9zs`A@KC5^+;ZRF8{0E5tCQ3mm0;;!w>H>Es_cJP*soSssdjDQhJr;( z!cOnx&Xe5mqs!1vu+73Dk1G&&&cW=;jpgc0`2RrQ=q=4_UV=5~w{~9djS6sZqI1;{YY0x%MewMp4Wula92hF-4*b1wH^tyj632L~QOSx<#klTwDd z61I$&3lFR%4D)xa?ut%dWgU-<5Y<~|xhXcXQr+Y71n zIq26z3ks+*Q0~j&r+QuX(VzL!&hgb@x8F(Yk`x~l%^w39YoPYjfU}v=l+$k~ykuur zgq~%(U+bKe#V{#I;*r9SQ%(luqa~H=FO}%^al)6fw!N8~*T21CFi#!cDIvBwX9^I( z#nHfsK0>}c%b}k)#($qy)=5vjLJhX9 zavSs*?9%%rl)hv#VSCWrKWy_>_W`nqeOh|&!#-o-8uytnsE6CSFT9cFq#+*eayvmn zi>uMpe{p|p@<#>@uSRL!rV@mHiXW2NVAxqxfVSx~#q`M`GJ}aErH=fOI zH&4Xz+}S)t5|!9H5+esUR<709CKmJL2SXn@>v^1M777}Fz7=9a*CaBWOMSz};CUQ8 zyat~+8B3}{@RxZF2^sMEzp~o9xTn|T3nQW21W+-~9v}|8DJXJ`c1I5U;NoS}L{$l0 zM$>arjBp8uYA)uubuEeSO&7gyV5OPaB}|4wA5hq3y#WK)s%Cuk%!P>TS`M!S)&015 z2y2o=%VpL;VMx?1evT1 z5$MJFNfwxfY;UXfV#*VY7#Fm6wNmUvm3`Cpj10MIIWdP+qmM!zir)S~pJ?QVA)383 zNZ7SI{6%{|@CZ$km*{tkTV~>K3=JZo=kw3laUl7`SKCH7bQMcpfz-K{Oo)d+>|3{>B$XrgC{D=?Z7q!^ zL>J$bZ0OX`gswjv@IGX<29Cql`vjQnpVuCF1wpLWFI#3pCF|y6nS)dv*7d8O(kf%7 zkKQAiS{ygTx^~CPZn7T7ZT*Gt$Kp?}P5)W07|`i~v-rZTJ{)+xygb7#|27Z8w$A4@M7)C9t*Gcm0IF z2MysnG@sO${GJ`!eL$l5Jq@)p*Rc|$ql>K2g~oNs6$4RM95?UFsd4!i3QLXmA9pxT zT+9G*?CYwm6RLkg!Wz^9S&caBJJUkLsp6dre~8HKo@3Z*461qps-O z2Stz9j&0nyanI4KZoD6+y9c+|ls!tkLLu;3WJInJtm&_bYzs`ESlQR4=APbKy$KA| zKY}ETGz~(P7`k8iY8D_~+B!F~`MR?Kj1#+GA*?KQzTJNIc>H~YzFjFHPatHHxLiG$z<1hm2i$F zoJ(YgRi&41SSn=&P`PW?3vtI6_APe%6_zs;b2!_N+r#rJD<$>Sj0_C(A7T;y>#;g! zNVmEie8^rXBR-zH#+av^A7&vq<>9FPZC9mH=iH1C+gKL8VHE&D1}_r)Q5cs1ZLP#O z3w;v$Xx~a5FJ3O)*j0l`uVoN%TsMee9bChh3SKLnahl2({4O6;cn9x`W5@*=LrK}f zekf-23zhTnuOe(N>XwWmCm!pz2~}5TyzwmbZFoB94?2Z&6fG51UKO-3ytA4P4`<7- zZUA7P!>fHi(ToICQGULsR~V5M>=1|O5rI5n!%tLwwExy(%T10L{H*5X`g8IhF9S>Q zAf{(8+(Dzb6#SXFDEzN4$E07FVhN(i6W+)W;^XDU<>S>%tZu3-3H zKX84s-?+-Fxc6wSt*e8~?5%B_%F4W+(MPj^ddX=vZdsPwc+F#NmCwFHdynDU>GEH| zdo0Z4Hvn+ok>!u|f-^2o-pFI8+PK9vX|zydlj+Cg>&$L)O!pgjUeiuiY~$6KRE(I)I%6ZFfz-25sNeaun1h4sCj z&?fUe|0k96?PrgUViD$T4N?N&eLd5?H`YJ8^@J(6;S26p_vb|e~R}XGR{ikEH2dZJyj}}RO*A1{0H-e3W{s@ zx%b3V*a`(YTtCeXelmS{w|s;W`{puUw9RWx4=>W=CW+t={B_^)sM7L`95{u=@rrG1 zDQCJg(wbD8k>Z#0Dhj_ex_LmV?O%99-7x_$=-N)du&J=`>t8GW*@%uMy?SXqenG9P zr~d#YFI=R>3;WKd_5kiUVLV2ExEH>Rkg2;=0_U8eRo z<1ohA{S-9!Rn_q~KD}~hs_K1k(=2k6)$Hc%`%OxoreEyF;H~V)-5V&_W6o0iNEz}x zLR^6vaGm*!04Ai~GgyyF_ky1c`SSsq<|uh$6|Adhl(be4Kw!AR<|>NIcDZ2tD#tTS zXXW=(xgRceD=7jESu_MTPuGS{O$C$MD*d~f?k3UVc$TQJPkS};86Eh4B3P0VhIN3v z>ua_D&ilW;j|F!g9XWF!pVKRH_m%C9yUSnhVG>L(N62!nV06b*zu)aMFJ#{Dbq0l2 zCUs}x5^h5pyz99SPv=Tb91VA?KU{Z9PTm`m9oeig{3Qdg@XAgX14vx`4=6!`N)Re% zAZgXd4F$=PEeXr@E5#cNuUu5)itf6D3bCtW%`cJf|#m#RF5{f68}vG0AaOU-1U_#6NII{}QY7^D>|y=X#6fVk0q*6yg5VxCO2p zSXNSnm~Vi9)WU5Jm8{2cSd*^lyO-6n8j^AIblV5nygf1BBD=pm>A9u+zyV8kL@0N* zC#8I%I&q+^RrIDfurp*`uo+W!y*!WW;y=lJqm?y8gaZ{^wQ}8{`6&Fer$He95n#F#XGk01czCaC^JC{ zh?;HIt2U^^qY%NmBKS0k#ah3d%eOGc^^2Pq@A88a zn!@~#UQ)F?42}eM{y-xa+4nvX%nLq5CNO+K5hS)25#JXw79zi`153MZ%w$l6syf5?mZvXwxFCeQvpq znON}g*6D+UhnS-hx(eGFReE=|OQI+1xF5zKVv}k_wQJZ1f1fYBmAk3TQarvX^o$-r zO@bq+Rl#$`!ADDk{hwUHN8Ba&14XG@oOPv>;A&OjXRc>v^1(BeuhJf?7WOTM^4FVtBi@{GF0e_FnFx- z_@*9-LG88wz%iG8es+8i%xShRu1D6uZ=g0;)!Tia9@!aajjtGv&VfT;l?380(Yg)X5XLI)&Nw6#Sf{WkZ)bFQDo__|qeJ1)qfD{gZ7}7-P@pmZjHqOUnFA z$2^*4AF=>V`uDw;f%Ln}d{!RpEvRJgkZpj}f~`Q*spHEyk0rzS#bDh4&-Oh7i;mq3 z10Gyi;qmmT4VPU~Eg+GxO!;(g^}8P9Das;D{&X?vwL^pIz`v3ALHEcfp$F!66Yj1g2}Dy2ck0jDixRZG8dN5WLOXGWNN?PT)12|B88>7L7acVzS7UOdhM1}qnqm_@%cDT%<4I8 z@53bn-Y|CG6tYI6jhw8=W}2?Ct>3 znJSs4n(S&cqLkx@IqmTSz8RuTJU}>@Ay;S=usmPv!~ZTwOuKR;LrGJA;K=8&4@Mi6Mx9XK^JB;!3sT9z$;KihUnjC zZ2cq+O?MArF+5TJZ)ylIpQ|%EDv8NgkwPpzCE&D6-@%({S=IQae;G&e25BWIiRHc{2$z1A;G?5> zK!-Kj^sXp4!9bRr++mvLr9E5a);~_wzV5X0m5|u6GdxzFJQ>kb!6OZ2RFe57;R87C z%d$^qUte(h0mvVF4cBR6^3K(s+UXbKT78Ij8VeQm$Zm9z9pU2RQw=U<#*#oHtFu=HOT_m<6y~dC@MRAXBiji&E){v&*P%;S$0>kWo)FdZY1mkJ8+j0dpRafE za-%zP<8N~@2oB>*Gr2i{_CdKn1)7=ZKB2z@PhuC-K%brNXJsa*_zTvR>nK#aU)hL! zTs}8cxpp!^MqkZJXXJ8sA&DB&>wyBk5e%d5Lf>ytYWAX3(jXNG1=)v4Za>mh5u!IZt5~o#2ZfEe@|w){FsHSJRsqOk>umE zpLThf{H6Dal1`4hk84?MG5nRTjUq}@7g3GZWl^}G@2$(upp#O>0>UKTqPEC=_QJp^ zhG}^kcauY>NOhL;XLV23Ab@C7#h*gt@0%hlBJg$;{HO5T^XID}OtE3HBF(z!|CC!K zmYOHa&CA+DRm$4EQ#yAhtDc5a{T_Q==B2Y=Ih_uNZY3PHU%OiE`y_%%cuP9DpJ69! zw+1tOWA#%6;fIs2-b+?v8vf65a~ZtGA%hkBdq2Mbk(5KcwD9{BadM6`WM^l4@iMne zwZii~_u8P+`FUE9uXuHL9sEOd{l5;0 zza%A9`ne5WE+@J=s0rc&x_F-Z*t+6)du5G&l5Z}F8*c>!R;I+~#7pjy)V})Smbo&5 zt2CO&+tM=I6FzvIUBu~ezEn9l^$gI%e6_&JcdO0~4M-yUtYDST>_G_BT?m6_lx#-m z{%N*}Na!y+SrHE=ZTfW=;)Um9aPZHqnw*VOW@1N40S!J^>KHZntW;lRrsH~O(QDO6 zz5kRY{=D|dioAQ0s1%Pnwr_%6nXsXB z9%-V(*N`(enExqCU!=)11SvklA*9=pXXywqSmtugD~S>QJ^?o*W0EsZMyEJL9~#EJ zyve#Y(8N{<)eezPp}nV7STw2p>E;rq$jl$TAM>SzEU5Z^^_NZjY1gFI(RcO}L<;yU zV$GGBWAu!PC;n$;@pE}LRnnK5|HffL$U>^aGRrp|ExN2Eku@xj_Ox9+h(A6==*y88Q>23@<^t2_tf z>JkmQobQ?@D!MPk&{)rCKH;<%Uw(*|y(7w@vTvuuTFR&;*V?2pvb&i&;Bp$zSt6Gx zubv)CD4Y9vGgox`p(eCCbg>mch^x4=lkbh zsKQOl(qNvs99_X<`0UdiXM|99ioGmJo9@OKNqbd)U_0#t$r7Mmx9?1_5Je5*+ND}v zul`^XCMxBq0lxcmDLRr;nYX=tDIr(=J0VPldr3+U#M6cda@@iDl zJUwE<_m_Ke|G@Ma2U`O)oe_`FfaymP$(~M6K$>9&h6a&GonLlNv_W76BTi0`6 z)tBykvQ-MD{&Q*ph#Q~kAK+sW|F<2E;wKTJI5b8NBrkt3#Ebsl<(-@`car2}-*xL> zVpuEFY)Oo`^pBhjL2c0FGy@Lj&_4kT{>XX#@f z+sF;h2|RC?$&+0UeXZ|8vg?@`SuWdyqJvXnVIz`LAq$@8yE#H|*|_^kG9(~<_6KD{ zu9%JQrM2q~B55Bk#6%motbN17BR$l+b@>?xI--r_+SR0q>`wa!F+o2rygDP|J$~cg zXN%v<`OW)xIe5Om+aCG*l&pVAz@L9!mcWM+!{v)F{`=Q|H;*f35Vg3Vush7BXFb zY4HE?ZoDt{#@owuD`@n8?%`iF1--*FI^rHL_^)i|pE=?GSjsRp-nvQHM*K_L|61Sg zPfykHTYBoX{Eta>|8*(v2(yoW=(zp_NdMQ;{`n`%SNxV(dli=czxla~S76z}MApBG zKL6L2uHX^>Z>SiG{mZTVla)Ugj-Mn)^gj{(cgXV}y!fAp{!c{zi}w6O_y1=_|7S)2 zY1{vcAN!x9|DU7(zr6M@uOt2cOGh92yaMol1SFtzwi1sW;$?C3BGeXhLtfghL~SCf zteK9nx}(qQU|?VG5xVZVi_&1;Kv{79_xVs0isU6Lk9=-rn#+fP@LMQ zj~`pW9ttNc-&JN^jqNzdcXVFdWg(l;UTO9#4_yyNOuy?&Q}$wQ?VrA)5T&=>vZ z7$y9hK|c2$9H9(MTSjbRLqi(t9zq-_>jyO3X?VHAS7Y3&Lgt(2MwIKGfjM(uDUV*kkOi>A82XFP4?%TmiJ{<3;&>h^3{3V3=+S7|K zPWQ#=l>RoM`|sxM_MKPtHXqt)6l$3c7kKqrZ*~N>IoPU;Tf`M(#mXar2kbp43bO^& zUEsX5j}7_qhuu$~-CisfzxxECZSUQmNx)Zg*01;Ugn<3D3Q)`r)PK#E^wJHWZ}!5h z&CUP!oqU$A9eI|imhZ9+owpyjXq4ZEDA9BZILMddAf6S2nPzl7yk@>YhGS^;sPO7L z=sh}sI#c)+j}Xo3;T5O-OR3LzI_UvG;s<*Sd?tB?YtiCirh5lj@Rx? ztZ8K^^(QLvBpS{;E~X)&+h`ue$7>(cEIgDBnyUIQ+}n`)tSVk3A8hJk3S2uFXH(65 zd>AuHFGj@pJ3Oy_ir+@OVdfZ$kp9>=?J)^0>rIb-=Uco8bzeiDe@%7$9mbWh zyefEYvg~RqXf>E=d(by+0X*?OlDBHpwH?i=sIbxQbToKw=ySTGiF!HZq1e|=H&_vS zbB*&w&*$4C1%}pMb4Xg3_g1uJ)q|7>s$SrnkVsg8=GIxt8>xu$JhFl?IVk5tmnW%W z+Zc({+^j}y2=KHWrzw|A7D2HJ9vVz_VE0wZV_WE>`Lq&iIv;VmXZ|BPW{k1At1ny1 zfog*Ck0v~VA$Y`!Kcx*eXe8hJt?Yk4#=_}p4tK`{hKxKdi3jzru=kCx0jOqY*!X34 z$zmKEbBd@BDW`Er(B2t3(c-1zN%oM4m(6QH{HzcCZ1o`zIl3(X^mP;2h=7Bv=*!y#T6a735 zH)yS%dcI%$RB`8PnDtcC6ioib%*w+7?r1eJ4S83f=I?(?s`Eu*bz2o;c|j*VH_pCu z!)!u$1YR0Ek5^>S$YTbI#?6>97H7)>JIzuA_bfx6V6L9LeC$26R@g+9UL44G?GZwV zV=lvv3I>?vQ&>ZfAJ3HW%dGo&$<4A2tbTkLT)F5}R)C^qc9Ed?WlBH|cksmIZPZKV z`bErFM5G@-Ug`_oJ2%p}-C?WhAmk_F_esrmeKdROMZ!l}+?)e0WU*(FX-h0WqvUwk zGQ_Poi1L?G;}enk1^m`V*K411uWeO$m72Cp6-V^Pta;gyvM;!34wwb82j~`>Zj~5l zV-_?RPo(B_$X{1$6yKPWr1UOGs|4{+O?$Uiom@R>cr9gJDS`diZb?a3c2DGYwy!&# zL>3pJeKUBj;@38a&qhQvFL!CUSmt;}`mspD<+N(7fq~_Rk)rh$o2z~Eg|~PblbB+c zD6~3z>I!Lz7TLb5c%2dXB|&d!YF?~v>Jm*((MdT*U+tj4TEgDu4>Ojf2LxB6psuh}L4CF%Ue>#}6JB zHaqiMHVB!$fl5b8_P=J{U6wbVpR>>KENzv^r(zQ09`vcvHM|k)!n*T`xw@ObS|E1Y z;;@_XO~)?@ySGWAC`jpwU9=$h5sT><`=%Bf+VheJZ}N2Zt+n$rj`3Em9M|6gT=-BF z8-z(Lu6#R3b>|$9qjq_QBfOvkIB!a+daf#C{t2Jjghd7Xh-N%8LR0V1xb@ZrElzV> zi@B&dzyF?;0>9$#cA4!m=}mBXgO6bH&%yIQ6%AiXFJW}Qqz{uW@Jm9_7KOUvsldj@ z^pR9d9FT$xc==*XJg_#SaK`1;BKHiJhbuonkB*uJV(%rmQ2||B4ZoNKpIWM@tp}@z zC9EM3bc28L5#MO0pJDx_a1EJshxr5Q$D@sz#c_g%N}*)QO~M&!<$f(rZ-`#R`4a)0}*^E5AsDM++9jQZtj?MTvW zs^Q@fS(L#op`AMqITqfU58n>X&`2NQBgA7a+46p#_4{|^ySQ*Jaqf01N==?wV3arT zq1_?*k>__dtD~+bdVS8`pGrGAMw2Hzp1=~B@>NoE2Xz-ck3*a%T!j2tU z5*Z)GUM5!=7-G!}`?Y^aDP-Ann7ZtVR?g#X9eAo8+2r*rPf05K1=Ryf9%K;`ANMN= zPIONyV>81^Egf=>RLRjNR6kNW{mjUvB~|+BW@A;FK#`ifRQ`>j*$rKKETSq3tN}}G z3A&%dQ|wK8r zJsjB#+xB6l=t5hqwsvdPZmGR%Z$<4{t42^OwO3nJd)HQbZ!v;Utu3~YAZim4Y9|t3 z`o8b;d%nLw?%Q>q_c6}%xRPS_^LafM5f&Y-RjkpHEBgr&4MOK}4&5k-j$9cr^@G*K z=Ar{Xb@j&|`c0qX+aLzxcWZgDX?@1rQNzt*82IB}LQ7IQm4=inWHy#8p{sSuKLn-h zyTOCEN;XC~$pKDFrXacyb4#y)xFRoZ-7bhmzs~Y5BMw5-oJx)DO8ZxNcS$zK|NE_w z@4olFHKg@xl=H1XBsN1Qi3v@|> z7A~L=1Gx3}1Y@aQyb|UZYNFVG~>ivMolZUw%u4B9BaUq3qZiM~orYF9{{Lpc` z_!+&t0^#`3E3p!l{8run+2kbRMETx(2hejOzL;ekqJFoa$q7Nc(RpvJ@Qtye3`G$N zZWiELvz9gDajBOuY}!CV=9V*Wy7gMQx*xz~8RCNfJf@m*4DmxKVxKsVdZT~2keWC3 z)G=JPgdImEfRfDxYf;r}p~RsIiSR}&Q_Y9mWT@`%#vfNqgQJr$)0<38qpos+iZgWM zTGp9MprFhd*M_ZsE)c8O&4W69_W&{RNB{Xqk%aq@3xs&)F}5dcBjy~v_$;#}Bj?;# zyL?m)kOA4Ye|tjpr^!PO-OP99KoK7Jk}emu7;MRUYPOTa2y;l_=bVL_Mhe|&mW6%P zucKd%-Hd_u>;508D9opoyB+pRt??ai-6-l>R9}g#qBCZ^jh#url`~auUh8x?m!>2i zaA;?o|A5X>xfzjRYBf<3#H_YDyS4icCrLgNzEv5C%S%<_;NS0Gobv?}rgF765Zs7d z2UKf6rQeY2|2@N?DyjBvda@-X=fUYWK9W5uQcCypN@#gge$QODfBP$3jb+QHprh3W z`cCb1a^6$*YA99;~KsXWSJ;V*sjOoj=)HIJ@^d`6xaqQqfQ>I$5>%XSVdVSBjsq?r=i+t_a#~Lf{ zexSW(-5!OK?ykwC;5o|OUhL)gu9YcjbN}w`=i@MjwD1<-62N=+!)P(Tj1lLvORh;ve7{E&``iyix^!!H>D~>YJV1cw!`=5ch#Bt=SJn+SdvqWmSa{4 z2xK}iupO2@I*PRfUxc~M^n!h>2GnYJUWOo%ubHkUe)KhlpcEP~K3GGSU0L6%5{Pwq zx7$Me{B=_Zht+B`uUW`o&$D7pO7(_=$ffs$*kP@&kw(LUFhk=R&k{y znfQrjKw}7lRh4xQTC;O@)v~&G*}y%M8`%Z7%7W=uT-(cKT0EMc$%KSQ$q>HZ(jl2o z7c`~4AUP*Qg9z^cNXl1N_gVyFXIJ1Q-e zzNaX3p6}G4!Q#!%d|}SK@9CUP>ylo{f8oSGd1dncxjtntb%mE^sa|LBLnf4 zn?T-;GKimKiRYqs<9*Te!e#w-bH8h@8M4y41hc!mbQ8V^hmPuUM;n(PNHVV7GX)Ra z$Db3u|bqu#|sAA=Z((7b-_NxZA;>hPn67^ zgc)S^LG-FFe>>{B5=49qTM&CALF@?IE>yjWYPdN9EN3d4RnNWDXWH?z$5A*;v84)H z!SR;&LCThD?FTJ42JZ|1ew{6+`Gd3o+LEkGCr|J0gD6t_ePoZcU$!Ziy55~81&cEH)s?0JaUWV+FmLBNd*|?y zYbRM;bi*{E`<`M)GW*lM5a=QYm&H{=XF10wK7(Y${S;~(wiWY*rn_Y^GY(bTs#>MH zpBHIfwP<-Yu(;$kYj@iox4(k%MEG$G+hK!!r{ttIMw}2A-AJhygR}9K=HK z-&tC<_%Ap5d26PUo=tag!UPHd#y5@0#hhY6HAT*#MbKM_Ta;_M)d*mJ(Cf_CEMaI! z^-E$|%bOla&EFP^(}o_4%z~n%i=>F17SXto>j3>y!*j{sMym56eB$n6tc1zQ-IS}P zlBR3}M`m-XcU8A9=T_(x@N~uGQOPOxW25T-Rz5Cv8;!5112(L_S0V_6u{}lgTY}M$kO(00(E;~_~LvkC^eRQbq3XqFX3qPoRuxB z9yhnH&ERr(+neD`m|lPg?(WUqBubfGodG~$E)nmW`1mzYhHBj*c__-Y%cF{axduQ^ zHOe{oPw#V*wLo9@;t$ks1#Pqlv|VZ0pthD8P`N7}v`j$=aNxs=LNY)N5uY(un%dyL zls%H@wbr!qC1TB!fL(FtYJlV6ND)jDChTzP?jQ2Y#s54j)n*@-(u$3M_`X^?N(smv zaHXhfEcrIf;_Wx0JJ)nQck_#l@Ho zdDK5Vn;`h$sLAFifu2#0D}TnpuP&FkkahPdNzTC4gL%HYn%>)+=yA8P2!2d{#`Yc5T0ZPe^&3e@$VOtw{Wa(U`)eJLR)!fzy8!{IhYTI}wv znhLI48W_v#X#n%FC5Q&XNPDF0Qi=n?f1f(7!k-`fT2Fn7NH64L10=+MK?bbnR6C}V z4+u)ObR5~?67n0h1z12mJ1#$G0AAb6oBDU1gHUwy5si3RehIanJA0Bk6TVvh&58Y z%*B}x8n9IR1$=O#5|jALRM%K4lx3inPfR;AL3TbdDo}Q^c&SA7?>{d$J%5xHMcv<6bx@`lz}FMeWOCV6c&%i zV0p%x%&_obcN9>=x5BdK({VQ&M;HN}itZ+J2=v(G4 zx?P2VQYCM!WfYvW$F1AP7a$#uy^^g?56kvFqgS7mF&UeZZI7+TSc4={V$Ky$iQ7Qy za{r3+K&Aywf!PsEBI?cG9_9kg58r(*ZXy)PnW(DePIMQ7#jT~`)oDljxo8VbnCQmF z7*5}@!fl!X+=bD)a9B`BEp#2pV}+gmlHYiggt_kaB5JnG&8RLFL+0KMRui~E1khh2 z!p16F1W)M-dRGEO^I68+wGJ)jo)pGX&*nXI^N_vz;j zpqg3xMtFNVd_-T=JdWEVm-9AAXEq68n@x2wmtNfTi67mRTZ9Mz)v=21B6c5#9-uiP zSYwd3n?)g`+%}6TM8wMViY~)H-)A-eQh@{GuggL$LB`N7E(gx$C6dfzBTC2`eA~V1 zg9FpIP*q)5IFh(Xl{-?t$VyB_S!{BMyF8v6`wJV~x^aDLR<1rr z>_6u{@izGsLypT%n((xl=yQ&ylfWzy&D={Ox+zfw--3oMg0zRf+!-x}K}vu+YFa~c z_TeUC=cpyjaFgpWuOu6+n-}gj|EpM0zxw?ys|^BVuVt%!Z;6lf9EJXUbs?5w&}|H# z>dg>4u&$`vBY?&RTQJFU$v!SY#YzbEXxi^y@O1xd zw1-F}KcJbsh*G9S&AqFw7Mduq79`dz-;AFig6^w-GLRysX6Z%qJu|gJZ2qEDC^v`Qs-(M67N%uy_ITs9e7 z{CU})^KtMiWL)QmJX8E|&{v93<;YHk+$4k_*$kza58`PQE0W53wxFwtPi8GUrO%4@x^LUgW?Z(AD*Y!qW7d6Y4deccePC(iQ%3vzZ zclMRjvMKb*j$EBPEzd!=aUDm~mNu(+^Y;>h)Y0Hx%!K*Z2iz8e6~azp69O>PN9gQ~ z+pW$@>2k-XUe@r}xOiBj!p>L5sQ>2@g~#49IRN56bS|ja;J8}`BJY0d1c}&3U;b_HLJe#I(!lGz5^l-elGsoFgfcM zQ*iS8T)M*$TAUYN(|mcgbev&rj^}vKmKu`RFn!VBE4Oo$oo~&Dk(rtD07cb|=#Dw| znCvRKwj_9w%scLnn$m6M!~h)=n!>K0uG~}eco8+W*UExhx!Crl+7M;R%|Kjf0pU_X3@+)Mqd#M|`ZDzXP&F2Gm z0Wx5UO1eTeYB#-NLYSLR_f)HLS#XUkt#S)u-zTb>+e?1{VL=z2tdt>SNIlYC!n5UT zB?Q+?>jmxBAuJ|}YPEiWO_Yq?3=mVg?kzXY{$Atb|0Em!6O9sC$lWH;XIX_8&NV}> zBF<>4hrdyB!{KOuZM`l1+mZt*U*J_EUr*2LoNN@UA$7(LG3XLnR+=Cw3dng1v;hU7wX>@g1tm81UUN2q$(8vmLF*&t}X#?Y%d~WUDOn3RsM+X#vqyK?@+6!^dTRJdvt8uj>z3XO7*OB6UF@Y> z_G@fDg4l-m45EO^2rG-i1q(l^roO6Nf0-;ceI|l#4|!<;&;K5$<_?v$O8;t_{xN6x;l3~Gl&Y5TNbIlcnbNz@Y`k5 z5?=KM{Pn*}?ct1f>oJ@~%pPq2EZJlK1|jcBW-jyTSn|LejJvn*-?7hU<7p5%X&qta zl6Zww@UAh7gt5TgSQ6H>Us!2ySlBM zP#rr(fVqa*OOX_nEdF}^i+5fFnwmKaJ%3JVS}uL?-Oh66e*GgX-tRU%MDUiBq5>9a z`YazKbi~0M7aV^lC+2?mIT2~fy6r!turk*4^R$lu{s}DgJ9_!z997%7eoPOg8mAfX z@S-zi0ymTPoYGvmbX_c%cvv}#I-tyI!dsO?CCoJLV6+&{ET#gQU(wNR^Hek$z<0Mf zJepp)i6##@mYeWwJPU{vD$9y1Y|Iv`7mSnpPZofvL0MjDvJOVYr88d0yiS+_EeYV% zVZxK#6#%FjK_^vG0aFYbW2lTN6n}b!0AFwsz7WLFW`r0=>~$?v3zNI47}Qk0)Cap9 zP(H$(N+;yW2SiE;h`-#h=+}Zj6aC(&A_ZTeK?oYkKMQbz{+nG5FXFN(Y5?NB^uOq? z{BWiJhX3ShbVU@V2_57r{2#(+aV>%~&MgG2MAdlotbK{2vq=*%tLb|Qx^=GB+4M=J zbCIG5F4jTpCG}1?2&3Z@0yNjQ^{BFLDvk}@F=6_XSKadUOlrZhdQUk;vgCEC6R4*K zUkrmv{ivKz2o^N~b6EMg4!>Zya6bvbMTF4}nqg^^jRLJeYphv2f{u z^w$1@&XOBq4eX4#NrIa`y!9Wga(!T%8|5*P2QSv63y6Y$yNTr((X0!u+J*5wh2@xQy3d+XE!!k{a;Vd(0AW*~O4?FiGvgTkV$ab%v^dQLqRsYI2kLrt(g`+z_X zVbnxYfg$OAs0!)TeyJ}jPVYV0UA~99#&~eXq$5kEvD-Z6QpRnvz_7H=$;qvd&#Xz; z#pR&z4SI&=&@YIg+Lood%GEK9lYV~iz0}?Oy;0J|+F)}tMg~TG&;tmw@a}-$jdh1> zxchkv^!i4;Fv3`K{-`Jp2+9gs)~H&59EsNRdU_h0;q2`#yXZMikga=5gM)(~;!j1d zhcpqW@7~ZsKneTs9(b9o$%fnemOoRj4IiSRTP4GiZN;WtE1|-B;8_ZM9!$$ai(>hK zRvo6g8rjKawHFva_H0ON$pa_rEYgRjc*lhoFcjiEIocLrGJF?lcD8$VA9u&mw_kV~ zXjB^+{C|QSF(fI+vri6+U%ijbMdc?X9A!_u%P)cMzfoRh0{6C`2V)od0HW9W0qXQ7 zCm`BaDF*lO%#Nm~%ea7|JCYE;iLKi4kmc2&yaqgDCZFlAP~8CVYXT zGGx0r^l3Zu^e&^hlqTu`-nF7!bt>6wVm@cqbe%%SqH)~by_6Th*#!z(3X#~@OvSB< zUTi1)$;ES;<-*&On_cziZkB>~K;EnHX(>Ts#sP6Qt*wtq2F1tXJhz%^Q|oJ^Fn8<{ zZuZCT_Khzt-abAv8;V;v(&1$QpUEZEm>~z-E2_y+KaW>p+6YO3sx3ANa`+Yitu=qV zuW*56gA`R{m(6%Gbi3M}5g2$%@wGJevgNSZg4=FJny7u_3h|DKwfg@S? z*LtE!ltIG+*cfGQv`*ZYXhSSS-2!Hk!xlw%@^f|MJ<67!-*lr}4^?rsxGWG67)6=~ z{;VBCW6*-Ip~!&c@UdkET+jVVezUIeOzjRuG7B+l6R{aizL*GkM@sK`8ycLjt=MDq zcof4_&8zOF7baX}_ z62Wh?qs@*iSuFlepWT+yvT6t1Wo4_IuJ$^9ihxZo<~{L1O;w3~x$oGs8OPcDq7z-W zEBP4D;D-NMcUalJb@zNsB9yfE4}X=RTCNDPyFHX5Pr!IQ5s)=hUNpyebvy)mhqPU0 zZdxANeAjfalp2tZ9;ApQ6e4f?VYKm9Z&fKH&EThjEz2KQ#<~xj*c+87`z`-YNN-7H zm86FfK%5~9g>HV)vwpb088teN*HBd-8(vRYQD!Xy{mB{_LYGCcIBlJ8LFT82^`iP3 zPAiMppZ6Ouv~zobrBq=Jy$eK{s|AQF7v)|If0KuZu^H5vG4l*Yq!Jv=E5BzS71cD z$14+z`BJEZOM8}c7Ina4(n7X6S~I=m5YkVNhQj(Zw_}XP=osaZ|5BWN@D?6^F9WwRFue)+@uF&j!vLyAPb_4h@rBiRkK@Y_&+mUofF z;VwJbt3!IEtpMz%!kpV`2BJH^ThYyEe@F7#P@^5(zU)T;&4|>ENm=94S)%Q+mU_MM zrqVy>RhHJuS4X>H&p=fDx4t&GBtr6)V8+g!d}pR?Um}v1x_+*bF0~x>YW!UTfebCG zd6hWpSnkK=WsY54A9XN%^us7PGw}wc2>8fwKi6Tx~bYYSZEy0Q7X!LPL;*wHB(zYnu z0>H1Y#}O?$ounS)^X110Q*CJ;Sl29}s2j;Q>+=drv>``sGVw?TQ(px{21#&p6r7Bz zR^=3AAqGZ@=7*pg+% z?GmUlP^@w$HytD&^N?4kVRawr?;(^u=18Wl_PWtuCv-ZOsa^OibMJ-sxCAE2;4Byz zVR?Aa_9F3U9%0ZolES!wlwJ6p-of6)I3T3__5=*F{GJh1q=lCoK#40sk-{|2k6rHX z+A8S~%!3o(e^m`}PA=NJ$Zoxf`+eo{*-0CjLJ0Fb5b)(kvup`j8G9;^UR2$QD64gK z!m3)G3?Hvd;})WSWFqo2-v^Ylf?wvj{2AL4{jS){d<)w)8A{gJjAvN5X4UkDrna#) z_=R95**RJ0*O&+j!?l+}EgzC8!$;2rj}!}quqQZ{S6*LfPA@!uv$3<8??NZbYY1|q zhaIfMt?Hr^23*k3NID;DwbY6?BvJbwPRGm+`5n)~*U@WD$Str^{4=b)A(Y z{FxG~91}!yYYc?JsoMznx*g_JU#V<11H~hnk^`1qzo~-iM(*`z=d=e{ne8c*8V4L$ z7$$RtWV2W(?~F>RkaM4(Z)6!y>;~5+KZsBW^vLF3D#)E&`zR2kKsIdv;5{MC)j*Rnyb9k_lxBUHj^p=k-xlBS*vM>t+f4c~yetlD93YxUBLe zNVU$zwG1iCQ;k8b{mnA;`w-n)*WjL+Eaff(R{p*fdN)ko}CW0GK+)k z+F8aai%RQSb8?)+HgS#7Z-xJeWa!H7cGwD}NXx0|ZL?@9{~+Ed&f-9@`tXi+Ja*(< z)wWfj1)b783uyJGq};KUu-IsZFBc)#w5j#*Y_KareC9e`ogZa9qjaO4r4j$K|KdVX zg6zkFq7z9c&*~S!LD>?yp?}g(=ZX}~;wVpiaWz$+tKUiVzUukWdv8`DF<9cg`}s_} ztd|2qOQ@-gaeiXiZ{Sduc}F0VTHT{uC(@ptO%`rS^E7%YE;-d`Np7KC^;IKSaPWM< z{}B)rd@j$0!2NvsM?nc%CM;7a+aU1SzUgG*VAGQ6HC61y_bW_F{-#nBC`B{YN2~5; zb=L2U{D~Q1S>pjKLVrhJjV5>$#t9M@Cf8ua;&)gjs5ZYM-LRd zH*&$P^JJ>M=M=AO1q_gL4xwhEwf2y9zJSi7Gy2d1Ij;o%8r!8?1f0bKL-NMk%>L-3 z%$Hl3_K;Cto%i>d!`^0JW)1Xz&LFLLCVdWn5V*QqeT_eQYY=!to`m%7a|_ao9in~_ z^t-VAbRavpur8>S_PKNXiWnqkLnu5`5Tc>gd9{?IcV!$I*?MslyZe=^Nxe3f{dd8w z4GnOlW6WJQUWa6kk?9D9qO)rinK?W6KUKcu`kcDVe5Vpmpl|(E8{!kBcU@yHq*=?9 zHR@vplZ@(4PF_d1C0$ga#+ee-n*2cHubxN`rT@hKM)1IApqK+&>p|E|7cC=PGM?gw z$a#6sRB#83`bn3sbl>JJV|V?;yy|?)1ExK`S7GJybJKwdcQD4oQ}>#-gN58iuZYDN z^z@*MGNoML;>)mx(k-0}FM0ZKlIyH`YS&Zkj~XafHD&bI3>)8%njBdMNuMRJWqop@ z2~Kdk%JW4xu4R9}A5w^3bNI1waM|qP6u3TCaXM&j@Uux{{c_eXV9tMgO4PK!vFP*& zThY^b%3lfpO%r=_&5)iUChv#gv-L02*$D2P1<2&fV>jP4E|#v7lj?%7vLlN>5{x`4 zAD1e|6xEaStf=3p%W2rmn|LN9*Yg|;0L~yn3tzCpWm?Y5+7^%o zOfvRs4rCJK;%$CK`LRFr=;yQ9rf-Zrp`9}_eA8s|g_l{3zr6~Ej}xjXB)!bUMBbH` z^Gi6;)_jv{H7q-`$nZ)J-iPu2`$Gf%OvvE-6z^9zI;?>M>5Mi5`<8+;D^s$-JeUT{ zyd!%e%qIT4fCi7QV8Z()EH_y_PBq9#O=GwMUxo}`nw0jx>9#yJHx~V}4tF5-xVhuI zj~8wV&=MK@u}+(_(hZ@HmuTlq`=TQB`d9QC&F`|nL{%LvvhN)UL$6*u5j^30NXmV! zJie9rJ?7Bu((V*4l=7+bbo%AnqRS%L+z6k9^dagFX~n6%xhAtu8DY&LbVA=D$Jo_e z9r{#maW=EM@VUYGGo;2+mUH7tz{=axZhMgLGU^f-uRtuEF!{-+dUxZIh11}|U`~$T z_ks>1ojHhIh#nkxcd?gnCnc9VObesJa<;P_8T~|Pl1J+0G*ZGofGc1nS+s%kc(ks- z*r7=6i!NCnvP0>3--?1QQ`?7w)`3td(`BwEu}#8q9@`q;NB3*S;K3fci4Q^Eg4*WG#Z4 zvE*b^EVlUNf<5QJ-+pW$h_~YXFw{^|ktch^LPCEaXDeDQIlz6c;9dF$r2>+fFSzgmx!}9-KiH@WWhheP-SVcFdk1WqDJ@K*&Jul{&x2H{BzdH);w> zHJ#UqHnq;ssnZi;NeA6rArV-G>0F8LaMl$bHgYouGw1JQWKSnBW?%;Z=2ru+GJVhX zOe8WN#uW$X3n`O3vntLt_|7JBJnCk@)HRJxQpTwui*%KvxZoxk0x3hiMctgjOk#C& zowk3YB3)!(q>L86JM%gvNcgD__Aq1aoeuwE-i5kQC9lt=`b!L1|Fk#9$z7VavHJ>* z+*FhpyI+&$Ej296lBUe9n&qPTek$?}%x~6Bj3rjWrmW*7m+>kgNbn=6Kk>UlgVmF< zXenB`E^(fg_SB&NV<*|UUBy@FW|4a^i)xkBwB^&z2;K8SyB?8=P>P(-vYA4hG zR*=zwmX6k~8}u1TW=1+>e~bLyMrg(71Eim_is4hR+5MkWr6*=f9M?*mwgkrW4FlV({_KvH@pALd2D=f3 z?A)oyoL{%LM{0Hm#Wy2Ku4_<-=g3&5F)}MRmR0W1ON0${&h3GDPH#eqUX)qG3TO~vJPX%1eQ3NaE-|7%rTGr&5*Z>lz4D_ zFO!tL@R0@ieYdUfdwoF%+FpSj(15SgF%`RwL+rLiXnR)~9?!*@X4dNtwI~ni%o?kd zQ`NexxmyIa*z3~l>UUKe{hHmh0zjRcYmp4cGRw_6)Ha#+jWlqM5CA;r-Abg@Vo`vP zKEp($kI;y~N2U8;J=#XQC>T}zYd$#0jw}c`CPyQy%mIil&6>v;dqC42V)G`sek((M zN5~qjxb2meZXosakp-DCO5O2rSMIPxakW(jvkZ}mWJmjJV;*@) zvo}pzA$2&p4)KDzI-qFf4drJ2`0q+(EsP`dHd@V9y3U&|pH)BgC^_)D>dWV8tQ;@a zmP&=8?45ZVsQz)0b}m-Rikt+z42cOx#e=pgl4YepK^c! zXNOGR;_TiTrORY&f)jWI*EkF+7$w{VQNp5oEVIdmM%lYDyglNRUp`har!8FkjI{6( zjMG}<$6a)-l?rDCCaS}UcirgZYc93(q?OWG;(3w<-bNRsSd^?7(O9?x_j>BvhsmAq z@GT$5S2a3ff@GwwkH62Ip_RnRpej?c7xD=Ek38zAzT1iskaD(@nyv1xX}qbR+ZYzw zVPEGzJx<_E(wxx0jh1H+Y3@`}{Q^Jx4jrz2V`Qydm+4gC`(~#6fBzg-l502Q3kq)9 z2WRtcvrUE6-h6IuOf9`IlJ6h9q(~2_r;^dT$`q)o&Nxu(Z|j!w;Gh+ik_nJ8W%HsR z{}4@OTEsu~EP{SOSqbpodbS|CItiAi?HEmV!wX(c9m%}1)y~E#WdNx9`J&5^(Q6~W zA)r0#6#kR?G@t}ftr84@m+p5;mWTGm zTT6oX#w`r+5=!ILm5+iB8?=Scl_VAp(BIj4eT4xY47UQ(G!hv>UVEH@och^bgyENL z8xJyv0ekRvBf8H|)j5Y}jG%eAsR*w=ot-O#O&cMDFBr-*;GL6;kV7+wQ0_R5;-gRM zzPNQg|IIIcrg?voTLy$~$g^%;Ur$XVGdYdVY2XsMeKdL1uVcEafO5XZs@kDp^1zd& zCgCb*ucpG!prc4Qq<@JGv?v+n1tR%<$5u{`CX%RW~UJL&gM zf3qRXGkl!&Ycl$d=TT9=BTQD$l8(-4jT^DE81DR2^g8C<6P*?3Cgb@@81?PmpNF_| zXxTC%{ldvdC&^nJt~2b;wInT6s`Zc`CS$QnIdFb4Xd0}O6XY7iqUui?mNW3?;6pPc zqE;6Su-{u|=7((+mbgY3SZ+X<+kxQ6;B&xt;z#IzPBRJVX9W^UPg}j^Uv@cuHj07g zaTR}_>eI~2;Uz*NO?$>Wd|d6N{00-#^_trm<&LB^ECk69X=Wx*y@IwCOy>Gay6ltR zq+Mp@Xe3jlq`Wc(T$jju=9;}y{waivK zeT%=PoxJE+;wQsaMfoe)YcLD|hG5Q=tT(B?IyE0GSiiFWZfqLuWiu6icb`mkMyJzI zcQs7$^1~`3-n#aJE{+EwECxDT`PyPz@CSH>zuJquPd_-rT9boO#f9}?1ou|Lc}zp= zqS1#;|LNc%QW3l`_*4gTpy?mK!mtF<^`!x~p`4v;pVZ|2^SCH34!YT^;MH7b+r#tY zvoGfW9{*C4cQ>hqDu!QHpM9m0p{)q|l83TS*3)*ZbmH1_g`AZxRFOG;0fKp%nO%qV zV_LDyVLWeHna<**q}VIX20Z9DIZ!$8XB%%y&-!^uy6OkZ%E;|VXi%^GS(}caee0s= zHHTS(y=*Z3*SE9fZ7@tOIFr{4D+h)uu7zahT^`dpeUhJYe(QGSUm{7!SCE&6X6aR+ zE>vgw54tu|IdnujY3M+^mg0^t2&Jge21~2gvsDp44{C2pdc})KI zBj6+BgpSFkPbG`RLQZE8$ptH{kBS@OSEeg71B*8yFVxU3<$fyZhgbbR{>TqcW3XmO zm1|c+EQO`s2h2l{g$_}V47;l7rR+o5FF*`Q}I1vaoGS>kc1 zdMD360QSF~y#59eqw`H=dorrmTX}03fd33NM;)cOKiQ#~WYRhbphb@fI#H-5rfW`Z zvIF0#aNZP@X&Yhw=+G$R8;pDZ_T2Ck6G=E=Twu0zQEvKCJ7aD=cI!=nZt#mSl(a}( z&2%i|^MtP4O`DmsrPb5fGpbB`qYV-xziMXe!{8?#mF*`VEf9yOzkya$?n&H%l9THN z@#~2Du%SP63t2u-mbSsNzpwl=+TX@44#U9Uc-L>h$}S1O<3%9WXw z8lG*E%*A}@>?{!a#c{8O^}2GnE5yDOY5rh6ar7|DEq^b)<**=SFy!#j-X5Rq!PBx{ zwqg3(ZIO!OK>AIxIM+^DuLU7fSHs_}Oib2i9nU8e&(vq%8pn0%IbrN)&8+v6R{A18 zwd~@ljyrx0t&bF*o`3rH;={)@Ss{$QX)ZGj?gLic{!%I9sw_ui2!B74wR7!|mmx>7 z4THmh7hiL}=M6kOmUQ23d@((9`dT;ioQR2R;9AQi6 z%Yxsz`;*?#8hT^u_!}~?ikkDD&h5tmS0-tMW&I&CBky-Se~OYNgkLWp#dzTw$Ff_UcHPHum9sAsWW?Vc1syPSNga~X6zmd1l*i9HK}M;NX-_}8Xc*BcYxpY)*_HJ55-kdRHjB2v@jb$zI$I&rdemt1v z7Lbv8c_&W^pW3%m=ObaQe6ru7ICldv@iF=8_tBg`?+CDWwb|Q%v$so7`y09?8~ z9VP!@^d6ICFJq*NWAK~V0Dc&DuvVhonNGs7M#>L!Cv!_GKl^dYZRt7dRa$Kw+ssC^ z+@TpDnsdM@n~f4~M2TM)oDAitg`NN<%6$cs)GM(u3eIO4=U*H@Iu4KImpVxB+Lhu! zrex-;tqGf&{=bwTycZhT*m^rIHkpdY4Lg|@^|oLXGOnDLbZ}#m`O?qrHwG}gQ3v-FUzp{#iioDvn0OCE1&l*MgLX?dfSiH?KRV>VZ0*0{#aWYI6ej<908z zwuc(WBNYQ704_Pa`S9Gu7_XZ{tNP35jc92$zA`#;pawu)p)Q(pHm|!D`@r;uR!sM= z;5i-Rd&yaPK`YgbC#y_UlivA$vp9vq9YV*}MP#41HKK^ZOea!lN$mMJo3xGnT$p9w z@w$j<+IZpmHFQw{!rreSW8@oA;0s8DCm9%u3fF3VAAYAiz;O9)&I}dVm1=5;lqP&c z(&Vy;0Gt9On*V8wsX~gKN0Mf>=z8G^x6;$kkYOE2)Ks6f_S7CX2rS|Cf%p zXg??}TG=)5V0cIXOq_q$61vd0nn*R_WOeHys+dGf|3AXn(|fEBS`+0xc5NSQNUZ#U z`jw0XcKvB7J(jPo*3}-;y7z4u8{2v-kN@nG+7#L3q68>Y-J(eMlpWfSAPXjwa}@u&-x z6;}J0zuWRYHL3bN?mSEG8>uY1fTGB8}?V2Q9PSV0wjk%_)0h8A+(SRMltS<3ax+ zdrgq02>uyr10qN~anGemX*z{C%D*+c8vUk_MbsQn=ZvrsFCvC{6K(_o&|BDH0AjQ8 zb78}UfD_(*1mCV-LVN>=1x~DRB5C4-^Rp$1JfW{rP=bwqbD|iR5*xM2SnmIK5xizXt zpT#(s>>RS_P+jCE?Qpg6O0zW$Qswrh%HqbSx4En47O#!Qpx<7DsoxKSqhP3c02Li(4ZMyZ4DE%YX&kDHb+t_SjVswed^J`3mdT=N0__M^_CHHU93XnH{VaZkRuvc#(DEI%%1A} z_|kFjG*UcgDnH9C1MI`;o)My$B9S3w#&iah1Mwnttz0=PD3^Oz6nm}&9^>dgX2?Oe zM4y-&Q^e#jIcM@b;bVsgIxGIGCq`2Jv9A#bsHf`hP0_T`^YgOiz`?$Z))PtJCy^1T z%P;qahPFy&v77q_tA|`|^s%Io>l5kR+dAQ@^xx6w-Vj}jToK;q4RM87CRI;p?VWh! zXb!f0+Q*J_U?fIBDr?W57aV+YhcK81KwB7F>aXPOYPe|Jp?LeEvRK92<<>Up!zSjb zr=EUU!2aY>0;5zg9=~$Vpn84lU`bF*NIQA}wR^3)YaEa{-S-*Id}i}!&7F~X8AupM zYKlaE-e-HmeNQG~DQU#d+xNcPo$&iQE01YN-pIGR1oPeq-ijAN%Vb1kaHNwBQ_pJV zU!r^ixF|oDp92hAgSb2}PIZ{Brv|K=?poncn2fBf(dKiwF~$~$HsLB&d&CcODt0Tq zP`jpZbeO)>^-(`yBEH|zN-M)Y9x3=CB)kgEVb-Z(aVbf@Q_hT&5!7k4p4z~j#MhW@ zXEj@3+Qt_r1a{8Ng3L#WeC&U-^4l2y*w{GelM37eGdhFhykAwd0E- z1SKdrYA!}At}?SOdN=eWw7}qNqOnq?)jY`Ej6>;whetFsc(Kf&@poqWkiy6RkG=N{ zimF@Mh6O=HB?u}aQAr|6L68iJqU4-&4nxiv5fKH+VaOmk=NWPm$zezX3eYSq)xCPH?#R6&n)@P}AmlpU>+rQxN4*3J(C`;TU3s~|9uMy2Bt`whk=#&Gv5bg8c|1gsabEMx5C$X z=cG#v>((lqyLL{L>gaGGsAtEYRd+DeoGX>27_}s5zIdkAiGXjK#-(WJ%i7vc)EDYu ze|V_;fXbdLxzuroM7Xf7m=YWI`15hgWR*dvXU$sh79 zzOjAluF$m0z4E`Gaoue6>C?)*f?&E%OQ^9wu20oH*J(V@_~?{~A@)-)ONNnAk& z?bl8*a5Ivy9cAjhO6o2DV7|&edCTbM4IB*%R&{Hp&CW+|-}-^gY33Civ+62I>8#;- zLT1Ja%uBqDbVw+jy0x;V>3KpYM{d8Y3hO&{;>SDe`bYi?Tsw1qiPX*Ybnlw=7VbWf zQ1mPh6L?RKoE~yFJE8SOmy=$}x)P;AVnZr@`ioK`!seG;5U29#-ofUz7?l4GI-r5MUH}q ziBf7u$t|VBSznqgy?=#_zY$uHG{94R`#FdT?D>`l_LB3gs4TT*_Ix8($4jF#!iR4Z zGTYA?SVtcY)=T!<8IJ8=e2qdeh*t4bgt^~mLCPnN^cs%ErZc}Sr#?t3&y=H=c^XV$ zT0L8++)V}xYu9f9hv5tVh#5jnyVE9An4TnMFw(QWdcy}ALD^-pVn8O8hIC?M0**DZ zDECd(K0i-80}-OeuPeRNMqU+v+~*13cFRvTgBrOodM};(&D01^0$B_!V;KStz=lZUrxh z3GyIzMi<}EBsW?j%6Fx)T9ONT=Gav{~Oz+n&n)0JmX5eAMg1 zFd?+{kp@Al&<2G9vyX6kn_#YaLL>!)w^8*;TJ zh3+_+f1(zQ&g8m28d_rtbpk2NcYpDEWrsBL2IyUPJjQeCI8MdXUZO;5;!%R_uVCaIS#OO$VVN$Y{6$Fn!_(<$ysGtg|o4~}Q|N?P9G7Z!}0 z8aMTP5R{o&7p!e;$UUo;)KT3h6QCqcI`OmqYyF+ROhvQ!Sn}@2EO{LihPv()0#s$ zQp!B~`*_vYja8Cdf}o&d^y1mf-dWo>+WIl1WD|ntZMe38e*+otcLs#M1>nsFxuUDD{yTI6a9pGXu7ZCgpF_ddcw22# zn)|Vhvyh0enlfwxh4Yj^t5m}*SfOE<#prvQoL;;*q@wFZ6wAmM zLrnM-i637qs^M-#yb1lzGYn7-yUNNs_+1s1YB?7>u|l#8qdb$jN#PU-J^hHYHkm0+ z39gsfBc^oN{d5_4M3{Q=ox<@F9Xj0E>_NslkQqD7pt|$0#l`Y5U3CdT)mImm!%#X{ zdB;U@bieB0id5OUsP#`}@BNa3hY4~oKNT{uCoTiT;k}fWDUE5Tq#QQ0q)~hqWjm~2cRNyji6K1QIjx26zf{rmtvR93UDOaME81C+0JqlEKH{i z4!Zs6Al-*`miZ!HYC0;+VKx;Lm0u4--854;E%?%=M+AVu(`9LQ@G;1Tq{vv^nx@XIqramQK>|Hot<<_~6$}kS~0hbdVjQad- z^ww7O?SfIa;VtOId5JFEuX>P zFOev?LRQl^jOh+mO@*oz`Yg{u(_{^&!#hemz&*TYGk)vVhbIiF?af!y(f5j+>qeNa zaG`ma$H*qafkA^V%aK#lTusoHSCsnFnksy4wA^?|yvOYUWL6?4kjrv_>iN_-J+l;> z>@67@bZdr0t-!HxAB2@QZFJ+yOOKD+sms%ygB%fvR ze)9N5O!|hhTR%?Z3&%K(To7%P?rC3RDZbYi#kx|Krld;X^NVeU$BY~%Ly{i~dZbZ> zPS${E*rUd^c?YlWkM1mFzxZM!yo%8eQ!>NZ1np{)i{}DSo9eq^MwCHgBvtNP2 zj9WMY_6(pYdJ{v%K3X5M5Y}}yW<)cAF89;QRH5-|lIfR)xAfuhF{1}M;Lo#7pmDD` zIgu$|#;-f3~kmeF~BD`{wB z*OM_8fpR1@@6ZzdM8kq+<;Vk5_Pf@7=fAY-E4AT$Qw$o_VQ|S(y(}00p&8X*`nwzaK22)bzexug)`LU6g(p9)_|g^tJDxx~ zJCAnk={rVPQcdrjGbRGtoEug!X$9RTNUFQ3N%AI)Psx)wZf2MBJ2D_in8;@5IM5=?mE!BR(gl)T*2Cq| zGOxj?$<5-wl2O1-P4WQ14PaQl^4C@cOc2#0dadT}#r&`uoR;jK$84a55c1^CvKxC3 zvK!AnGP?mldA=b#gZDnz@HJ8eLkJZSA~f+@htMX+vvBm+bXFg2>D6<6M^n>`fFK2 zNN@zinF3P!8NU_J)uiCHG9kqf+yYK&EgCA^qt4;E-9Wvc9Y$aHxevP8mdJZ2-ZoRM z0;b^C`?zixs`Y7wMDPd1C4bQ6lMwc$7D#YL4wai=SN%N}SE>Q;eooqT@{YLNnmwS8T?zR4ioQ1iej?2$Pzi){^}6Za?`GEEJ^IoL+}^!DYUlL@n^)w_4|Josgg~`# zg66KM&p;FR)a{h-1*ddJ+5Sp6ST|3J7!(-B(e30hbu#f+(6-h%qyb$G=IXCb&dyc~ z-dsnmEKL#Wa%^h9dqf4)d;QzO1j)_yfS^!@G8*$}O~0YLs?V9U0dIsJW<%Cj@h_Y_ zaiqG;F#l`b0nw85CLc?(4^l4vu|RhSaUo51wx*-zW!0TJxn?cB0Z&#f1k6|Z6^gD> z+frZ!+~*7D&ft}joBeX;Rle9Cr{9O&b||}hxe<3l2lZ@iex+D=mo#-huy7t6J`MA| z5VFXjaq37nGE@y;0Jv@oj7u50HIg+(b7Ufn`0U{X@tlW+SBW*{gr&(!yVqX|N*`%} z2LX(@GQ)1*s|}Ia(B?1RS<^yzuAoD-#rS@DA8D68mC-RT=Q_l8bPoM1?St3+yNdF1^Z{cGk?QR12d8f${^3MI2yx(!q3rsvwE7Nz zC8@u4hp!zRb|mGKZXa1ktM?_REe`NEO!M3n|}_5|PNJmfu_20M9+;KUxog%%ky zPbFHrEATBBG_J_m?LtXHCcdn~GjwFINfKTeuZNzu^cMYlc1EQ6ip|yVB0ZcHd!yFn z`+x1UJ0QR$gdQv{EdKr59hUNTKDbG0LbPK0(UeNE7JAL9vOl7p{0N%IGPy6nCaYDF z#mrf;U{7u2f0i0AhXi4?-Na$_70{m+x%Jiimnk`*#?!6t*8wx(a&wu&<^H{o+yE-{ ze=PHo(`fA3`|`4xK2$h!uWL33^=gV@^E-{2sPi!ff@6L(6+2Zmo#Y0H+!rv1G14s!q7M6RYw_}U6MaCyqAChTj8LA6U*NpAoTzbHmS-WuRz z|EfRxYjXkiAmf-+9Jk&~Y)<{JUI53sY=d|H{5G30U=um^8Gb6)f2CBbBt^u@Oh@Qw zv&FV&7m|h+Vv>z}5*u!|c|$r>GOiewQ&P3Wpk*as>@H$a&a_#SS}I&oU^`PRoqOH@ zfjP*wTy(IHjx~GU#Q-E?B2?zgDQ@B$)rS0PdN4lSdbLv!#)47{`EEE^OYy65CpeiNfKjt%!+{ruOwty@dluHjVxb{!*Q)l!K?_C+8x`$CrmL!f$Ievb$8X?^IdmR36R;rWPLwaFobgqg!Fnd`tTocw1_Znn^=Gv?8SiX%CnfkLoHBZxhUt5PL- ztjzSXw5OW_;C;r-GBmW2xoX8)fg$Qo2=oz^X3_{q9$$^j%r~JQpr6v=b-tm6%stCQ zhyg6fDCr)%J`J}ALvU$I<6g@w2U1T4%xOkJTcws@epWLgDX(b^FN!@dQFjb>+I_n% z(g2?|OctC3Vwd#&1idM>nk|egZ-tVtk=d0RD`mvmcz=GG@Yw+kx~w9!L_hV$c!69} zmB%SdgSnN06THlakB42u%`urdkteHaqZRj$CH~81{WeIC-q4@?Qi&3LBY*AB+KSV| z38}|HHya92Vtn2gPrSDp1hlAIm9Tvcu};D2@b7mx`UWgIcFl@c+<2X$2=j| zT-g$Gzdse|;AN@;Nap8qR8C zIqLfN>te5~J~o)ztSgT@@bwKLqO6)!4Tb6=8oSt_Ho86akTTkO%{t}8%rW>QZ{YrN zj$8fO0u8h@PF!(%rb3-x{y2kwVhpm^030Q(pLDtVM-X>haS8T|K}oyqr9vfjR^OfE z=K#8yTMj6j`R5A$`z_)-Op!;iAwQ88gVcXs@jI;k4>kYusJJv)D^5x@I8Oe58qYt5 z_WMt!s4K*C(H`(sp8xUV|M`i?Xdug&w`i85@()4&Z-0tQF#@NW81-FGZ~fmN{U06y zl!LmJ)l+Nr&i{vTD3V>C3i+auD*6W_{XagzDg~^7w7$9^{{Q1ge+?=2HK1iOKh^SJ zfBnyYMg3n<|E-h%)#^Xy@?W;gRfMnQ6ABOyI*q1(?aN1zam(6T_uTCu~ur>F#=9YileLPVv*ECzmR_rmjaS@wT!xEr`P{S3*6K&5?+rbXH$E?GPc~v zM@@wrBxQJB?|L|*HDj|=T%Znpm9t-v>90<_w^Uu+7!T4q_3|ytKWOe>Y3DQ2n>s2_ z9cS6lXjWD;P_KB}0A4UzP0RP5o9^+~LFejVe9OO{AG{Px=5hR}hRm!$`z+fE*o6{Z zM$d}`017{O8O8S}n)uJPh|I)`=SUz=X~fpWyl^@kOnu%vUaB8g(1kI;;trx@UmxQJ zjIm_Aj~EQDJ9A0DX|psEgg-SevE|XFjdpPepfxcGE7O8J;{P^Ls%1+~KvbSYq%$R? zL1PGcP~#NrW|q8-R?k z|4i5XbHIup3A?{Nv)}y{Il%VTesNn*(0*AGRzI$8TRfB2W#=Nq2fRc%LLB3(P??_A z0FU=BfA>-gHfldxJcz=_%9roP6=~%>aJ8L{eWqTa&7o6c9tXFborNNfOxIl|>-d~n zL)&n4ZTm`W(%2ZsEGMU#g&iRr`@RIz7%}8Z2usN;_yhc)&u*6@HGTB{DnB`K^Zyb? z&>s+X7YKQul_$h#kPSLrNOwNx8efQ|>b@Ab$$uCieuFfUeedvXioKYa##8RD5;xH? zH{p}hS1b^x)kkc4P~FJlgcFmjkOGW>4 z+)N)^3(Z1pH%PWDq8;^{w_AMHebFUTQ{&#$G4|*s4T<%MvZMNJm13{++!Ht>8h9E( z`pctRfbSO^V>8gZgs_RK>2OVgRGpXk4@$>9iL1HhR5wxV=s#s}j)gHLmqAzC38vyp zgtY3U)m}C2jW^KR(U6EBMVryDsm_2a_&D78w{KDHAVT?&M=*I_qiKb6+oRA^di>Nc z)CK?#`vH)5ZASM9=iiw9KXf?cN`TYdqDhoNvz^K(9~F9ws)d3#JQ(-!$%lvIIo<>- z)!CX8#kxjv4I4&8Q|>hf>k~nOY52GwB_Ar{5~>y|bwOva&oO&}EPEwKcMHd>+c!NX zdNm4@>}{DrmWc#EiVYgZdo>DZQ(Cf`JvvgW)uz8;KG;^FSk-O&I{D1DC!oeV!YbRc z+$={gB6RPmlJ>^3rRRuxx%A%+T}D{Ar&|S2A9_`P{+*!2m%L17eVg%UmnnvxTj<#O zNI>6yO#eWKXN!MawN?O80X8@)Q*+I+hLN3s#plW&gZFSDaGh1EpJU}^$jbH%hTZ4^}F094+Le(C^wqx7eE_#Ui5yf`^+mc-)lt6h4 zo8q_&oRaA@`4spb3B<{OLay@2_3{m=LdUtki;Fc0anwx5ICFwd`bkDKqK3!D;o? zy6dJ0XP4lZt`MI}`qF-sFJk2t^87o22zl+R=ADd(*JDj_3!~T)O@YI6s>r1cp%Sf{ zrWcUJYz$Y5K+m!qr3v=ztbz@X8kE)I$6)hfJ7~6TzQMTXg99vo#uS9It1i*49J=tq zN+mx#J<{~<#Z~WgkkGi3Ahc?)(_rqZNR=RR!3Ao8ec@XR)o6#p*LfjJq09Aq9#Y3? z?B4w2k9jWl*6tRYLnuiPG9o7bW?lj#^h%T*TUY|zhEmsx=_E;5OT^{TT>xn+S9 zz>f-ZiE%bve-wXSRNulN=8s^;?!>u@9PZN3Xjd7@ z=m6cD;yFn$zpUJ+zg9AAo|m^WnO9M#;FG?4iA(N}`F?LOp}I!|=>;<>Nyhr^jm8eJ zX|1fb`$Kyn(CVWyp=s@XetJ`^^~nT@LG+RXaE2Io;(qQAB6SWu(-=SO$ke{z$YI0s zDm_Wl9E$MUrbm{o_i@v&yE;wA1r~3_?z(a4R8(@D*n1iC{61g^I8~1YR?_0QgN{xHT8?JvL@pkk$!myY%w;y71`H zR+)Z{0QOeZgM(zP&@_UiaVaaqlb2A~IDYHK|NWcxD%R;-&-`0!?)PnK5qeUhM7qXQg&kBz-V<`ABn*_HmtO*hrFBd*UhES}?AmSvcRqd0{rFiDCxHfhacF!AYk5Cg2j_E4y!3a9VRkAh)lJmsYKn`}B>L^E=( zS2PNN6Km*mXxtvk%n$KXVD#0s&0{(YGQl22tX1!6`0f=X*Ermuk0*6=5o^SJN*<_|li3m(CrqRLeOqHWX(q z7g)sLbKD#u^a0En)MIde;OCWiyg`HW^vX}j4a+dW{e2j%dd&%6E(O8ASn(U-t(jUX zuTl$-8E$zVDS@UPEkNk2nSXfR0=W!n^ywwmxx_$`1$=gPUV3+`{Qdn7?-fDRsEXFk zi=a#Dd?sTVnqa66eAqmU#@JQI2Bfy0T>{>dJm80M8(WTRch2qz?*38COt({=-`4h$||{+dMTz(ww%Uk_~s{I#!&^s0TU&9 zCVdnGL@L$nTpR0Sa2ev4{p<~H-o9m^hCuN~RWg#s zBlmpwJIzbMx4Qz|_=3+ZQ6l_7R9;Q7pG3>d{w2>vx_~c%+?DKYL_PCHVkYS9SVT{%njCj% z)q4XsCyF)i;jYDbzVQ9Q#pgY(NK#y87kXa*WZN%Qrj0X+7jdngCvR_lB}+BV*ni7~ z#sB(am5YcMn8}bgd+6**-~_I^g`oUW|CZ@?rhsi(RPj5bqE6|AhjXz0fr zAq3yE!KWYbr&`p>x&&X9uW=K#E5!V8Bf!G8@Zr(S@P1k>c!Clu$RBi9{86%~afV2v zy{ph^8_vsXUL*DfQ!TUd@cOXtBJXzhE1A+h2K-)xTe#Vpcdwa+E3p%vU36aMy_onD7IpHSsoY_i5mUd_VMTE8YhZiVpf-Z~SBMY!mp(npn`R_{^*{5UoMz3E zXwet3XVGc!GIMCmAGZhESkwp(Ad$pf2CK-g71n8H`Py0D`nwMOxAoi9#zgCvYBi}m zul)!oefJO-fO4QMec$pX7jp`#gN~xK*4=`M7Mir`;F<6SVQ6D!p8#+&zsp8BkY*Tn<(I(6$WkPI<`%~#i)pj~b&)%xrg zYgqq7gkBzkhsDPBJlw{b?o{T4&i=qvstk3O9yQ-p*HTl7Gw_H({4&hRuby^@FHR>QG5)Y}15Q z<4G=vcJnYwV*vIj#V?aIEa(-@WS7A6QS5k+(oc4~75(x@jaCo}uxVI~k(lYSE5F8maWW;@(Nbx*+P&!6|qBG-yZ!kiBZf(Jn$9#madB zG$WXjFhTt)Tl!$OL!d5q-C>UMAigq`)L0K{52|Jb@m}slYxFc+vCxK z!!0n0{@j|YyZOc%vb#<<%WtZ-v^N=Unc(?Vqd!~q{lBfuA8o&r!xR(y)<`g*Pp!}x zqkpd2%stvSV@ENaUS>O@xx@5yVGG40mXYiJ#($mM@5H6brMPbxY%}7>G{NJ03_HZE zlW1b8A}U!(K&G(N#uB316@R{sm=%ZX_GEf)KnS25*80O{F`XQ9a&o>h2`n%f#U2fA zN7{St4$<2A&)XYMVNBxBF4cf3>OD0r!SwbkX5QQzoB{REibji5A*-%GI{DbD54^1VON+hb5TJu?(Ej>|nvv5sx5M>bcZBwmL$>C%S1ddGIIjZ~lg_OPJDDK>t z__07Re6w22pS{}8f&RLU9WD?Q$#~sJB^G)i^Zp~>n7!$?5EX1HuG-0=k=aVG1tMYZ z-&a^qUYj9@z+!YyvQy8-=!MRgZjWc^)nxAuB2v2|6i}A1^pUpsGq8G#M))kNL~{i` zHaCp!EoN1;-b+oJx%{-*<8Z888rkO}_C)OlSoZzzUXi#gtuwi;x1Hbrd?$_wR64(w z+Cpo6W0Wujr96bB*>535Xj7NrucWl#sdE?@r_VYTmME0Hjh;NMrQ?gZ<-!orUE1P0 z@naN`z2&6+xah5xjo0eG4uQ}ELPm+gZim1x>^tAq&~620vpH3ZP`%9HCym7O!X}6B z;50m^+`k*Tf*kRz3TZEJ4*wm=yf)l8+A{Y}k5QfsXVIgWvTa(w?Y7M3ud@)2nCKb% zBqGnMjc8H%n6I#Cj^9&CgFuZJc#(Px?nq8+@p@DdY|K#bIS-Wa-SuwJj#mPYit| zBG6aJz4Szr70bmkzVs{9B;fuIa%;5`Ml`iW+LGV8Qey*u7GvWqPn_@lN&>cAGFt3w zvnS=ar3&B$PIe7jlnKVqDokkOPBX~Iv$e@0=)@JI9t`8&yk3e9cz)^30={38f=?ei ze>Hwf-+K`zw)5Q$cLjXMCFlun@d{iq}_lJe-+)6@VM>)Smq zj|16S1<&q;_Yuus8Fa6G4lpivWN>d@q#Ew!g%9l16nkHUPuDe z@O;0I8L?=3uhQRLh#OHj04>&r!Q zj$WJ&&$etjc+XeM&$~Yxh5uwe-%%T30HVr~1a0v*Wu6TSJrhWTeRz%j%1n$4=0^|^ z5?4SQ=;Ev@*6iK+=c;AyZDU(nMD8NEm?Ar4-L!6`jzE6DL?3sq0o-NK%S4kqLbx{@FYLM5WLX^i zq+Ys1DC*x$T#)m$T-LEfNij6 z`pl~qJp%RsIdmw|0jq3s)f(gImbFEy(Je^n#g4!$DX?7#I z1#f@oWGQfEC4bma%#N1Ql1SLbe=$fvuq(x;x@N*Dy88Jgin)SIiN{QQD!d@Wx!kc&KoD_&g((okx_u`v;4#V5DZleML z_coe)apHTHX|s~-ibU3@yV&L)M2m7rL;PKxKr<&uY-g6uFYkzkediqdJ!;le7tu4H zhV2Cb=uCjc$Nr|(wDShc8_GeBxcW#U?EXoDO*M5RPuDHzCmOHG&p2G3@10`G!!Gi) z)1yZ_=_7C8RmOOMW$U?TV-TAZqL!&7VqCBrpaX69N@nC1EiRU%`QTuVU(PKhv%1&? zQQ3yTpgztYwqy0^1__a3^e=K(Q0&{6Ra=e>P#z40?o2nii*KM%#8)N%ISAe%ej%T(0_Nb)Y^z?DT1rjZN znvtP(yqrdd!yH}qP;Q($el@-zab{dE;?qJH{>yr<(U4j$(<@zVhCt01P;l*{5So&_ zlNyD$&(fqoh$!Fbt4iMToEjLnY)CaJ@=jZZ_=hSUZY6BZLUz>(5!tAgaumwrX{_Ms zM(Jq#$)-my(fCfevvK&7)EeuvaqF>>H3*t)UFCU}1!LleXd!?_vB2sZi6}JmmFbfN`We zd)5bKzYHlA6)&4E+|ioi7{}v_#i4dBE>CH-QXaxm(D2!(UsR6feul61=)9d|g+d%q-iv37K$rk(Et3s}H`HDTB*GIsYu2!A2K{ z&1|JZS5)X;B^yTEucKm{iYt`4HvkN-6ij>YD?l|e z>w#+(2ZsVxNhU#RPeMx$O5ZZtRU(>;m{)+6sB|wCeu~#WVOQyz>m2ob|BThdq;F&p z>rhnVsu82Cnkk3b>zd4T(#L9qr|pOxlECB&Pbd)pS8XjZ>-+q)mjy3`9Y=t!W&cX; ziUQgaG((tZ381J-`Lv28%O^QGj@=(BnN!_)%y21OlZDK_rdwDfi`eB`Y{b`SnFN~) zn9XR^(>iNizgM6vv>F|wTij!p%Odn^eNq3in8Zb(3yQ6`!mDvK2LRYG(3w;tzMEM(5jQh@p zmA=^iOPxBGZ*gTC5Fheh)vrg-l9V@F!xITQNW7M4sdjqy4tH!>I`7p?v2uXy5g!m0 zW=Tqhm5Z5iQpLsd-FM#A_!tdR{~8wcd6n^$@Jwh|1Ewj`nI@DIIB~_l^PY$AIuL54 z?%!Fw!l~12tjS`Lvo@A?(8-U)>~_Tfov`j%Q=>MKHpLC@u6rdq-8J6h zeE=;7+^-4mIZvV--Gy_lziP3*(CC`ApFziO@A*g6RJdLV26d({!(lJ@1J3a?GuCsZ zZ`6C8n+u4RSxWS*Zu%^vhbLsyflD8%zqBls+%wGmbgaxsSgc<+ExHq|E;YC~-ibYn zD!3S4E`M<0KI;W;-*d6-*}vs;aX8?%=_zRC_Th33-r+IddJxI=R{35zH|Xj0J7&0I zn7#LmFP*LTgh#Jlo|RBcx&F(StHD2_Xxz$>pq_lP3CbMYp$`@@Czv@a6p|@gst+A- zpX{pg-#he+Uk_X|4Cd88s?kp#)DPYF0d0`Moi8ZU%(&o-HWdpv6-$MvqN#2`iTfYe zjfy~h&R}%fU=vQc!i3xkT6B%@x7d-n8agO!Q?y;x_)4I-(xdG-F(3(G#6^Ww@!u00 znk+Co5>`c2Vchw5tdZys81zOB%Ot180pE;}bmeOElf1GNDE7EE0T)jO+>?)Ja69OI zpZ5x$G`_G{oz#lhIXxc;{;+uAq{05V7$Y^Ib?!C;wB)}$)bZHil02Y(M)24UZ`s>L z*RJGgN^7(sVk;?+&&xVw3c1qjrw(+_Ek}pz8!#l^`J!IPx$QkcZM=2*{KZ3Kw=4qV zz()#m!G04{jXnM=tHNNeGuWMy+PdPtXW+$?As`_$E!wy=DZB;y4ce8XS{n;YZ0B3Z zcXL1>AH&xN?=?6KqPft+*YS3fTQfAaAO<2Jh8L1FbJIh{I@iSwoymtV7|;sZ)ZUkuzZFbbVfHQ!l)Ja{oww#5RcKs_&ZXh+0)mH5=<|sQFT*C7+M9 zv_Ez|?Z68tnAJ`@HcR$ZAhvze7iY_1b^dr=Pb4)wlmsYvLJqy@(kJu~B$u9Anyj1T zWX0p@gZ;DK^$;V{$JTh)TK3+h^&0di3)knn2JKy*zK@J;&F3zAD$^`ccevnNhq~u(awLhspG=gdTWKwR zq&zrbgFZ^G@9urN>9MUfo-d)`J{=o|E~{?^NdDQlyzw9L&#nf1e-v4qTNchCwU-p8 z%_I_rLj8(y5xB{7U3I^BmX9;8Whzkju5faXuG}}EYx2}i1lEw{5QDfRR7TfK4c4AJ z*ZHEjDwT8k@ZcYQ3H?55a3k3A z`;;g=O|>L_Lw*S#oS+vpI{hIB>*Gn>ENUztzoyArhRjg((38}#M;JVLU#zfEH1o-e&K!LgCb_ z*8HB|JLLiSKO|G^Twgg#wN0OKtuN7vii%=R{AplN*FXw0-J`MYjkL$6H&tB$XL@0e zL+Kq>H#f;5H#avUNngHHe)(!uHS${2b8&HL!5o?ThNw~*_h0_aUD!=~_e1vS$tL#& zj3~)QzYveb;EBz&zzT<_>3wG%m=H^T7wBHL**ImpeY!~Tt=jyETMSym@#pALu@+}Y z9qYPiX@DwvFZqW;jGS#&>Dmbm+u|>*5S)z(%YgHOyw=`yPfP4jCUNR%pc3}tm z7Q7^@)$^g`hin&32l~TwTu>d$i+=EEGPXP(?{z1G&X;{ZRn|jJdX@6JI0Q^mp<4YX> zfeunC;UW0PGio~a=x-hztRtBbX5cbLaa`cvwE&IIM>}y6I(jyzSV5a(*M8fS zOSjP&-a{)053!a5u@R z7W9*FrRUlTBuBaNx8|d7z8u){ArkD%Yf0x;zK6A;=qPQPRET>VQR8LFmL=+8)V7zc zwXGLH${PJcA?{6%10UT4IvvsY&YC!l3#a0z4hf2zTIunnG)TwsF2F5}#asQDsQ1BS z*X8ia&EDN(aDez#F=iP>Izcp*Um|mlLCspJ&pq7dzT+UNe5%&{BOtB5$HaA5PtKM z`wQWeC~X<7#4bu!>E{9Mjq>`!e)L3_4EKKJocuVp+KBvnLFqPaGkW5)z zRa>0>W~PF#6#wgHWqunDos!?@$DCsuR0DCUQR2U@+h7V><}K&$&$sxukwoX&y$z<1 zJj%j*Hy`0mb$b)WXaP}RlHS++w*EfC!1w&v?xm?Kp2@k@!H~kzVO{$N2s7Sm&1~x} z%1;YDSX>hJ)0(DCK1`3R^CAM92UbWB&C6y}J+TFE-@n`ywLw=H-p_o93-jU$h#bGd z7lAOwbJL(vL#hrXypOehF+Gi+hxt6+hR38lX6d#<@5D{7E-aL4wlcYf(xjWjZI>{Z z?`bp}nczgHHi)*oL@yGki7M$AE$=Z<#v~;LQas+W8?!1npCKy{bApT#Vj&9+rY~!r za3A$KZKX^nZ*}+rvL)Mg!yR11c1lMBhxdBbvLJ4sVVkI^ABlTI$3;03IeDb~Rc2Ar z!A~ToU_;mM?di>5e~=`)t$q2Z2Svc6Pev?nO3B>vzMZL@l#=JTU^2=GCwb>Mgg#~2 z&o7J6c-?Hj@rc$a+m)IBnmlFLgS6YI8m^dw<3TlnMp-&JD#yGu zJyE8wsSsjC-KW@?t0W@=Ry!&sDkfmSj0C(9(^d#F5{QYCdjIOuHWZ6j82;pMifjc;xmjHl zKAw_hX=DRfzBZE*`towJ- z(LU>1xfU)Ltwwm8eQ4{OFWC?VWM8`un9&M~TS&-%S7#Bv#}YqK)7&W)FYlykH@`g{ zn=2kvqu&+CxXp=MW^e)fW$A6VOC1z8_9MtfZ5%D){8p=`?zS-Q=Tng?+b=24E4w!g zp32X?`)u=+^i2Qt7g`-#jSH_NiCBo>c05+mxlT6knVZF2jgaISqnqF+urngXHAxL` z{3_4r_x2#d(RhP3ZvDc&%-7hkcEh^EFnW$y>cEJVb#fonqf@5xIVs|{Sb^%rDa%Q@ zn~U91`={>MZzro-Fv*VudTyyp0UP;aiF)cvtN}V ztwGL2nUgWvhV^L4BTp-`f5LN5)Ov&Y9av%;X@BpGrdzvoM<>u)5GOkm{l4tkiJ_=92&-9rrdw_Fi)HLq-Ngw`_QtFJz0Z}vIVd_?f0T&OeEo%}QHiwfZ|cbyP_K27}V zx+1&anR=xIoED%9@OMEa$z-qjEj#s8oQ2)GmUtalcuaq{I4M)%>(J!x!i9gKfV((v zK?<5PiAN7{NEjWq))0yZPuRaUFB9Jb@wOJoT}8^E2&$WS1h&dUco6setGczGWf^Ik zyL(TyV5*hSl&M;A?mHz$*<4!`i{8c09!=P%iOxA(P%-4d>W66)eL zi(8f^7Uyarcw>2v)YQXT)HWi-j`sza*+8I8%s)!6su}!Na(}ajXb&PGpYSkG-M;AS zJ+6vaOOYQAwAPF;x><;b>6kLwum%fF)v+KcSYaC5`)Yna>ZzFP5-PK1Cb`QexOISs zz^v?cTc|6s3^V!I0SCe@zn#8_q!{dH??l@AJ65Oh&@?ckC3 zc9#d(dH~ZKVRt5~iA(T^$~8u1g`?;Kzvke@C!U^B@0!IQ^{5iCsK1XwW>dPR0LL1x zeKCtuAReHN)8g;18NLuVi!6LwZro^KbUyRV}~S9eZL{_(YsKz>Q}zP|3lSx$5Y+@aUZ9Ha~gK`o>`HW9huo=WRp?Zl9hExC|jhI zP4*tiJSc>aL`FukvnqQ%pQHQtJg?{VI)C)XeRIxruIv4Iuj^|wMRvO6ri>(zmPuz+ zj9JumOy}v3|Ox#@8Sn}E5#k8#cs?HAT=1C`4M1+Pgmv9kKpwAG*e>WU#D!ah5 zXU>hJ<6(R_^sFk$8s?l8Uq)WWh*KvO1$NC8y z`HCz*k}xLo-&a=_`>HQ0bdGUx?hq@SIni%+Ht@;Ly$ICD*U1`E7*Voscd{7AFQvbN zhsgiFr_+(^7I~S`xOuCogU4@XDl$BX!bRIk+@HpwcW!eOQ~z*%z@_w_tsp_oQ8Y^- zjYZK#=Ytpjp-5Z=aoZlvknqo&u-?k70+>Doyp=ciL z!<=Sn4h4}6o0c!4)W4*O*Mm;g4T)}8EBq|mNxu~K>35|IJN9p%_zg#|sJWt{&nta@ z+P4y9x5tli;uMWC*7hS>v8DlkQ=3&BgD<`Ly4$03?_Kqg0}F5TrE3c(4ayiUdg6cQ z@vA-@LH%93i~NY}>)E^3d4X)(-Gf{tbPhoxBD68uhn%@RwzBu0&Ed}ot`5!mcs(Pf zD#5kgcP0CP!FsNHwdFF$vv-hZihGD5e` ziYrcgyyf=3f&_sp1zl{y+#|hB^sru>4gcK*BJ&%pw@_qKjSKB7`74FXkCom1y@qu^ zPRuunI8?X8a|tvF9DO22WjV0r9&?_B@qZ)?3?rLn`OB{8uW?K_VA5InB-9Uox2^6> z{QlMKm=H_CHF}cPv45N4dTxCLtgS9;fAq1i?z~r-5QUd9O1NJlvoTyl!}(AyXY--y zn>hEZkMEb`1bo=)$~e`I0)4ddg4 zLr(^GUhecuxhc|x1j|&6lx$|UMetlE>W)?8vNN7@M_?}%e~t)iNb9i5w8mL+i1!X| zhc9XUu>0DlFMBU&E=HylU0Pz^kh0pA=3Mqct5b8lT*TrqjBv#fOY zrq+R#*F-VHSF;rpQSN(Zq7}1dztGY6B_!lT^IMfA1qD!hAN+ot&o)i!ZY8`TN?F~} zLpP!F*y))7n$e6@F!}bt#7?`H^3m87?OD9sIev}HB5YXCvy5ac3y1{Q7pbB8W%!lE zmcQ!8;it0?I;+RHEzElPUbjKWx9_YeQ{Qwlj~rx^ZU<0XvpnK(TlAgba-FR&_t|v$ zs8txkXG1lx<52iM@8@FdGIIKr*q$+aX2@FCo#;miuAQk9q)`XUDCX~te8NB3v)wYw z7*MiQwjU>Q!(*$gpCr@gr989c&&s&@Xz`oH5W;g@1woF>iKtTN_#VC?DL>by65qLP z`Z2Q{bBi?i_K6T%Ycy^y?H7&J_0JSSfN!jUeUxbgqJK{h2St;4*$!CaTCv0P_7e4# zg#5-|NNmm-nTjOl-%*q4#8AiinwH1kV_k^6kUH8pcJI=B?AdF)(?YjQgmv=w8}1bS z_INbMz^=lZqQI*2^PEQJh*UvN@q&8G1F@{yGOr(M=b657F17pLqs2~iEdNo;0#-)x z`Ne51{078Lpe5h=+|)Wc7#x;cylk~@6jaCxOokc=tCN1!VD~}fwm;Rqi>vh}QoD&N zXZ@VoPs*BJF@9%Hi;Yc+J?7pk?2`gteWzv{es-o80q0`8Rc_$GZPUbfN4{Rs&A_0( z*fh(m$Fw4Hmu9S|T|GsEolYS3{-^-2TSD6U(CNo}&D<_~``W`2&XuoCuD0)rjb6Ny zrMs|EAZOfB8A?Am6d~|X**PWice~>9uux0Y-{*2Xe)8)-SI*4rdxBQ;_WD!)Jr^h} z+f3bl2~p|yX04U?YA+w3ObWT)7{P5cL6)iA<)!z=EY-s6cGg*IGJJOf zgm;rBMz$hD;o0GfufEzHeV8V`U0m_>$=AbcZCx{uUS;i6HjFIgauR5=x*UyobJRAK zi5zBJ3r$qNc{!$l;a=PIq=zGvZ6BnM`K=3@(Ns#VT4RArbMUZ(>kZD_)*C;-_)3H_^W^NlDSM)B@Ry~Sc;9l#$p~=F$ zrvyVY8%1XMG{P`UMsVNNIaC{0V_4ODdZ$5-H{tvxpATk#u3GVqe#j+y~*Gvr5y>J zYx?a`&4bf&dm8V2cH&L{3ONxR{!=bC$V-U5-*sJOgy9IClT#CQoUU(3C)}T!Gf%9K zzEw2MkQgbOv;U@0kQ0ZJ48(QiL|eO-h?9`ZtTm>7eG@it=u|fKH}(@{ZRhd&V2v^I zBw;HS`ybAbpHX8YaArqvsK9jo+SJda*a>84=GPz>UqkA$s6#LR7uWbPNk1H$VD9qH z7&rdV#8A}ut%M0Mti_Ji`5M7fRZ=QmT&d8H*uJ6Hplr*2lK0-+Q1e>UrGDiKpN+LE zb(E>&4cQ_s3ekz*GlPD1>-q9Y-wCXE90N}-rW*G8 z4Ux$R+U!4<7Rm6RxH>Bqd*U{FFo5*}1!p3X#r@g*I>ozLTZhN^2`d?o^I;wl@(71> z6pl`dqR-%d5XWin|Iw(pY=z97bCMi27=3UZNKxPB%*>b2HqP@;H)#{_Xf|n&qOk4A zW}=`jiY<-&~$llT_HD5bp_e0;cyIbFPVG(`#=KOaRj-Tk)xs`ma` zyHO55ndD=rN|g0hH_PIC6#4qSNM(Su&~;e1VUuZ>?%JS206 zUwu*@EvB@ll^c{)`9>~!Z(uihG%q6d_fDM2)ePU*`x3Sp@%bmj3aa8zicf?kw`}+f zul^GMAQ6ifZK7mr3+PFK_3>xlv}KqAIel`s(G_5sSy7 zli$qVAL5^#!_ywy^);vCe@v(~P;r+`ty|rA6Z!S!~Pb|q1NH{Ha zs-}su{?QOcO~77AVUn2a;Gy)5M#h7>p8`fbmI8U=4`PK9H7nd+M_BBin>nweSgshr ziHU1-uL(IhMCnE5r5a`WjW_RRPQ3SWxJvoJLbg}zbrGj2l>5aj-`<<$<+^-vsu_yY zt0hYl1QU)IoT!Q&cUJ0_>NvtJy#9Q-zF4;^q(rTVTKQhbqxxJR_Cq*L*RSCUE~Co3 zmJjaq)_wX~U@-H_xiR+T#XBm~ZIL#Eg@z*IH ztO=DFsk}Gy<3qSdB#*}HTh$K3>@P)Z%D?PQ_hc$XXWe+lUG6X}va}YkJw<$PqOrj} zmEm}|lA@kc@v=*J{BogP+~4_=ZPxIR;6tfEN*;}lj#!Sb z^h4n^LIYl%i9$jFhk4PoCB|=y1x#wRG87~EKE1k{Skk!r)c(BXKJTr!PRUs+2`YSg z`J6Q8O=3dMFyyapJ>xkNd-t68rs80UIZIo^k7fM3WA#M?`Ia8@Heg{zWD1>byhZk* z^#eag#@QCa{-g1+?{%K4zkh!G+H`d3Xz!>eSt9O&mEyaAqei`#Qa&Dm+O(woR6`VA zw(^H{V{{O&~2uZYug}AhM-Do(W-UE#Ww2aGU+_j8=XzPJt0=7WxEf-K z@GQUngBQ_$y9+rYPGh1NGr zymr_m;LPH?7HIt`Nv}xTR`2Oq{x?#JHXUe2Y2RBC7jDF|KQ?Lf6|;`qUDGQtD0yCP z)781R+El_c6GqN~I-(Xd;(UCHW-vvL3%^2M{DF0zt15rzKXa*24PM(C-0Dl+ld|q0 zTlkU_O~t3HY&TRaSY`M1$L+>z0j#1P6l<(VzrD=_*+)cq%*_NeJ#Se2{>Bn-^j)hJ zx!2FRFj7?@(srjKMwaY5rk#y;QTS>UqgZ(WcJkaKc|_0OFwV!1UK%D{apY?{*nDO9 z>A;1%^udBoo?*osIiY;rJa*yvW4#8XNceL@Q^Pc4Nf%mF^-&n{(;Usz+F#O*gMFVf zm0&~jj#wrL7{;}k3%=~*)l8Ahb7Xa!?KWy?;*aI-c)9NRHD9kkQ;9U8rRc#z|L>)e zrd^5iQK>K{+R^XBX`(f>IP?x$CrIKnt!67;W$g z0{IjEo5OcVv>Trn|6Ll=ge#9+Vs5kXumYdA?FlH`N^Z5w($A~Ey28t$ouwSBOGZqafhH13Y_XDQom+OrGxD@6XDyvOG| z8g&=3+3uNa<$)paie|gsCXmZpc7H~tB5ChP)8bRo<&hifKC2DzEKhXj21+AYJlm0K zL5f$F2O%fEXk@H&&+7jAN1oK&Bx&dIddx#P$!Gr|gEDRgMfR@BbJZ-=BbrHq8xYB* z@8+?KS^O3XPSUHidKh*{{(bE_kLBA^T7v1N+AZkdT=lGt4?Y|FJ8te) z`^n1BxYfTs2=ZK~MJm&ro;xkS(~?&33^uhy*1c#lwjh;X z#+gnHuK5&Q($e9Oy%9rSpY*LwuwUK#b0mocHRofJ*^hwNElK% z_ReiIzi>3HT>d+!N(Y)%P>h-*X7MDY>414i&c!o0*|{f46PN7nTJ_m#G+; zFClMAOD-d5+r2=xZqF}_dgWf-{%+YA)$>}6!*R4)FT{M{>6z1~8Ch%W4SeEM>zM3fT`8>)D&Y$kA_VulWP3Q^bT5r`7Qa*qKSx zXS#a)Zc@?viA(QO6}!vgu%|EsCFbU{pI+q{1&7kvMaqWmrc=$KPEre;3pm(F+&|n~ z?L4)5c(50getm#AQU_KJ)Cn{UKkqXL2e129)%yICHwvNqNFdoDFVpxdLC;W1dtiMQHLh-wEKTHPIYCLOx zjb12J1$mweE6ygoKah7*iOx@O!^-3#C5!Xv)1-`|tP0`OU!$mEo2%XRwO^ejnrigh z%O`eOR?v;ZGN4(jBy?n2zf7>NArzmcvnxEkrR$5=cLb@d*bh_!W-Q_lln+`x0+EE) z02|U5q)A$~&=#&9%|V1D8{5@R-}9*-`Z>R6BnO)1MS)F!Yx3^ChJ6I=Ni(`d~W&BcLdWZLDX2THS3;MRF_<`ByB7Xel>h|5~P0?%J-kD3)W3q5& zrrguOe_yu9Tig8#*V1%Pysn05^&@I+KkS3FJSwxArTk<47BfGK_ivxJ##roI3Oohi zvwHAnR=tkxeFWXjHyX)y>PmK=F%nZ6lkjW#9x_x-3--TJXP}dMSSeRz(B<0rkcB}0 zft4UHD=RMj51eit-ZUl>+LMw?;{2L#BvNbF*szs*L&c^yZDzX_%|sV7XPsv{6PP-C zA8R^$8yt(fYslA_P!^x@mnh{)CX`mD+?{=etjm%Qc<~u&x$P%Dm@xX|hhc|G!L=G| z?~+AbGwOYJbUa`DTc5i~ctGEm&DHJo=^zRuC|_%&_n+B>=10T@yT04q{o6|)SE=_D zk>l#DFsDV_X4UE@UXothhxXxw=5z^hQ`NcEoAq9CV06;aq&ONvt|CC5&@WJZ%chZk89eCVKfCgo@3B0>1;b9? z-ASOtgv@;+XDa&Iq1H2U)xYB0X+WUKs-MTFiwrv-%cpu&L2yyZ*%gH&_*^HiKk@0-|d7Ag5s^U+-?ByA8Yg*wcA%nJCP;- zgo5uBLu6CR#$nl2*b!5z5bLhZ^w#Z>1&&nN^Jh8(*Cu4XqkIkcHl^Xa``M(CBEBI= zXSD(S>&tps4?ll|fk!xwC6h$^16Q|eDGoR5H(f0dr|orbsk9d7OaC(EW>8_PzR_^? zElP+@&tHz2TU}Zjui+GOIr10M-~*u&sn1kcNcv-UHD>?_Bz8|wOQ$RN)t z-L|iJyL=lMyn|=aXecR*ew$oD5QCW#g2;k)NBb$!=e>>|3kKRC1S2X^z_?n?FS@}A z`%4(VBWq$2H)ZO`&%fFfLf&His<3_eZXeF>#a|(I%d19`r!#tp>nlG|0EgC76$O-< zm6L8F&9sW3z3?kT`@&s$oooL4x5zBjU6}|(eOY&Dc+);1d5C-=p0wj`J+2DiT{5vi zU?;hih0m#Dev|S_k$-?cP2&W*0e>*b1dwo6C~1A%k#enoja=Fp^0lo4-UI~JCskiv zDUy$**ZY*9efwlk)gyR}M?v{bGvU)si|i2_0P1nB(`~$REFw)W33~<&D{RloyN#-V z?2Ly;kfJR|-QF#~L|JSreR2{#VAHEB8YnV0&?2TKVZ61J@$W$5&qS#d=}N3j7zWo@ zKq=oXe5NugV(CHdLdM0zA0+0!>yNiH*J-Sj{+2lM-JWO7y`eh28X#b@69Ce}=5T)} zh>&CSiIR*AHlILkx`h8;g&=-*z>CbV-m8o$1c*#Wn(U*Wprp&bxE<4jAUQ9$;K3jCE3DR-Ygd9=Ltl4#_4h)=b)=hwkW+jpnH%-&eE z8Q!QfxU-O2Vg2!kr$CDxZkuuQ>Y_de1NKA5VV6*oa*CvPOa7^P(*MER8G;IeZz3-7 zw(0ogl~KmvVlLSa3WT1U0L1Dv4>;h;(}BnIf2r@0LtFHCY^|8SbDN7|y-pQ`ZoU5a zq_;BZ@R|5qoi8qAMu5*?BV0iydvgBcGah4)WwY4W0;Oo?pxl+=iaT!1PD<^29$K{m zPGdFsgF=m|wjUGt6Gcw)Xr_$$hV9`umJISp6F#Lsl`>8NW$`P|s4Cyo9z)3;4_KLN zbk>8oW?Bac3|WWLH7l?hKbBPsZtr{OJC&9bgkSLPW1A&R z_zOlysudkE5ijccBUeO7`C7rn>+{7qJ#HA>jewosGdkHSXJu#6v@cj+)P1g56_FxF`>f9@jIxhQ^>?A?yOG9Y^)QGRdjT$8o9Vi&Hf_h0z3uA^MrD;tn^4 z%vFO>80(|M1G^R{CkHr{I>jbp*&*6g8&;Es%4`s6_ko<;7;cr4crc$P`6@R5>9LWR zqK0t)6Jr}0E|{`^en)Ad@nF-~@l7Hy)^j`pM`<{1yylF@6`R(@XL9!+w)9OFnsim( z`=VRqIuMb)O%!O;^*qn0yo0vdq8TasmsOhBg{=3rScUWH{&2C$AndU!4>Gf{WO4U6 zpdg}=wACryXDD0<0w?_UK3SHZEYTkz8&vl-by2-?EH<0Dv4roEB|J~62u`w4cdTzw ze!kGtc^b9HU5`awS4Kny_-w7{xnpU-pa5q~OVID=U@J&X=-32zQAEV1xX`yvh@oFU z8Y(o*bZ7P$B-$8PMG(DV6uXbkQ0Ps)=FFrVe@^@7)SWr3=m)>`v=7R)^QwX~m+1zK_8@uhqH z^BU<+1p_v!52R2=Npuqho@9m>A8-6Koy%c z@F`D4=qNw0Y+P+RY7}Y9QI2hyauP;oJfKLqwlt8J`MSP)rit-j)hG`( zq0zL^Zd9(gSB(zr7iAnJI*psTzvLz1rbkgHhsb5BtHEI{&{3wI z7BUJ=Cym94=V@n#7`21<64CiWVY(zLZVMj0W?C?)vi7d@Q=bJLUHMLtQB`Mo*RdpF z4P>Q&Y%eU$iyl)GgrHXAol-(j5@osff9y+_qx+b3LM~YqBOgX4o1Nbs#}zJDx6q$` zlhSCAz(Yn3NT`e`h0HK2*@8Uz_}~IYj?1Tkm(XA5>Fk(jY4WYnuMTEWqXJ%$1ogJm zx-T&j4d(0d6EmGH=D_~CfH%deWkcf7l-Ja+?Qj-l45?SBsHE-uX9Djy8c$V z^^?DgU%Mc|nc%{DyMmVD(l5N_@koBxP-Py2OGnyUio+6ySC3!m3|9jg=818a)|lMg zSPH7`+1px8l=9gWjiBaj)h=O}Fg)gMF*0j-TUgRe@)asb9&yjn0{h6|9vHP*rga{C z$y9b9CuzC%(elyErryyoKy~OZ+i`|<$mg@wClTHY6Zc#x?5AwLd)0kWcm4Uf6|45B zk1!;4pH_gdz5yIn@B1)||8}#a+Dhn&z;9(%U9jYOpLo44^(I|b&2H&;=Momy0<3)_ zAm7^Na5X~0s~F3Vr>nALjTP4`4Pc8&RE<71UAD9F-HM_2%Q3b!B8tOU#Y*l z<+MK0-DV8@`SIDQ!Nr*LDKK{i*Db$i0a3FP?8^Un-0H%Y3gm?s$qyL{`MpG-AtHg1 z^x0JNels;SB=Bdh*MRa&Matfk9O7QO<=EGc^Pe*b$MdR#Fqpr>D?a7%{016e!>{&z z06Co(FHS<7i@u5!9?T?((=Y?m`z)WXR_qdB48@>Czjg*5b&^r%LEMVmm|SJ)mz}=s zzBsT_&o#;&_?ALux9PI|pfc#jY!G;hEHn`lPa}wLfL5`1V!ypM)&JJnTu-Qfxc zMHG{ymzMp&_jhhNxXDwYr$x;lERX0tr52b)Iq9et~^ zPc;1Wk2?oikil`ME%oBVph6KDs&inMyI>X#acHG=2c+l9n0APz@v59^w6@PI_)9-N zRvF#QQf4zKF{^qoy!$lwC8_i_uD16~FLz&~pHG51q9KC*a=zNDE9x&)#w#W+Zo=Z= zk)g`Su3x2#tHuSMTj6OUHt?YiL0vh`QcBdeltb zlz@1>LGF4B{E6Xe$5$IBq9;J8%h81tsW{aqAj-3v#nu4#Naij(j_CUDEng*;qG}B} zb5p|v=y^YLPX89h!jGzIC>A-QPr<3onoAQIw!iqbV3hgCm18z3Ba9Iyd>|aS{s6L7 z0?CefYcT25P86M(wiNe@Sc+(x=a%f08s6U9UbiY*rx~y6$w-zueXY)lOy+?1&ulh-au6qI|kZ=($4&=#AI}1-}6kha&_++GDw)-UqSZRaIt@Y2rtfx=a zXIE_nYO1G2n6l8_W`3HPfKHiMeY!+5fJ#&p0%l=$nG-IwfwPYL(J~2qdhtqm-am6K zU3dT911p?a9zmJ~LuOzz*ZN~z(2aDSQ_OGNgC1+k$LdI?;()$q0}#N|V1PT|O@F)p zXL>fOA!f)P1QE7?7Cs)-oRh?;nCq36tsx`m_wGxBA7j|$_OZxNT1KSL=C4>dk*7L6 zhdTjW7lbWNy|-nEqUwow*I{xN(ARA<;FWR--oz)CI(w)(2~#C#qhG+VyaNa$=k;K; zhcMaB9yobjcT)XqbLL3?eODcL#1nC)l3*kCM^%6oVI+l5zz=0vRpQTy z`0YN>)6~6sJB!?XcCtVsRUC+hgL%hmn<$b{Ir7D`IXN7XUq z23M6nJX8T3?9OqUh(3;?07-4R$gVn>fa;2Hzh+QuBJ%L>-6iQ`w;Kqa6817y4$k^B zoH2$`)I}yci(kKxmWSM#Yvvs>`I*xkc951OB>sToS(W_QKul*qhKw!Z(&s|C90dg% zGK_#}t!{RqoRMlcJ}vWD8RG>%r95~0c)MC^;NohR-#?+$0ihz?bj^7_6;*mG`e7i1 zna{ySF=AuYLFFOcW0eM<)c_lvq(Q-G;yFP~QpsQ+%dUXBclYKSwR<^pQ2~@SK3m26 zzq^F`tLzQ<(X{^BCusi&cF!F6^|T-5=no(1Iax#<$WNUTlIYs0nI z=$sj+@3k0_Ty16Cb;OdjakYcC`&&V-A(%WoAnK}AIAAK=Z=O3es$$#2=v%~N9q{O;njvi$A^V;%9Q@w(TD z2u+vUkG@FHuHWm=(r%l8U7Fi_y8P}=9V-Q8udi*hr^E6{m2K5XXjzSJo=%*g@s(v) zzJpf?#s`zRR{XIXO8QN69o0iXln%{CTc(Z?tOkqtlT?FXe+5)z%?}A5ZiqhM!hLz- zQTl*%xuomzc@7heOm$QgX!1ay`nkH}FRXiCUG2?V--6@bX}<{u5nC;(5Ieq|;z>>` zc0Wrk>Cz2|BP=f;{-CD~KDME?uCCxSdY~2`!Pacu-2~zdG>=?H{x;az34t0zgagP_ zDW7T)>7rX@*F#8{RIGwXsN5-l5YS0@6sA!NXS`p93IJr?TGsj{X6o080zoC(iKP{9 zf+ntN2XUk{UGn^N0xNbk`8eVvJVmYh=al%;iv<@59^njVV##socRHxy-}^6c5MrR* z9VZ_lgI+p5MB`nuVHl(5iiK&nBMNtiI-Hkje{TOXXM7?s!a2)26STE~M=KL`I;-7WkIj>tMHPSF#yx zQeq~xln1ORz4PI1z3nxkMt))6rA~wNO=rMu!YCSB_+FoMzZUyaI4h00i@_()G3@(5 zo_<6ASLYU^Ed+e`zk3AWiABLB8`JhU!C09Xj7h^oy|ze(9G#nlJM1a*u_$4>eyPG1 zH}dcwmV(MRBcITJ2tfOSA=tScPVNl3wHW8Tj{6QNc~m?MRwmT_xt=1&=j1pef&U!O znlRk8qkRs%w1RByoE1~jV-liVmpYWd43#UT>)nxe*IIZ5DFsmyF|YDqt`^-H<`!S2Xi`kcr$tP%w-i+tfj4D*-Y zGNWZyN;W@-+`anJWceD-ES`KZ6xBvkDC> z6pcx5`4A*HCznvlf#G&VJ@+Zh!PA>k9V!1Zn6Nq$c zxBZkPcKQ+kL6M-O->xp7e&IPd2YT++Od451C^Zz>YecJ1r#s&6C*(_Ot5&%E;VBo0 z@q`Gvi$?!tX%sFe2ow=@1bnYJ2`&<-@#z44!kc~$qt&01&eOU-Sz}dScS2sc{a(vG z7jhWhGxV2Yyw-o@5u1L`6)t;^Bw9Q%ExVPI>aWB6tJX~}Lyl#VJYfBYLf{2j_eiZe z;u5L}_EJ#G3*!xS9$FCS16oAg{cn7;u~|eR+9sq2y7~3;b(NE*FLRPvBRlK@?`8&` zAU=7~1VzW7$w`pPhG-vwDPZ6J=u+R89L*u-%sAAzLK@(7&>TbfI?Y_GHVa*qaL%Se z^FxJHX}@3KzR%&k@zzL|YmIo=P`G#7k3fTl3V;9Vs`Gd*+D&x+^|b&q!e-X+M+!gM zBKbjPW~>h(gO6R=NkxdW@wPBG!KuKO$EQ>w(iW&2G@uQzM7`N2A{c0YDdAa+L!aU( z)vze7kjG5ad*75k_Cc)g+L!&3JgjpTl_M*|wL0@&OYu+#1lKc?4*j=@*_ z`^N*vbymZ2>!8!r9k$3-zpMwNAV0zQ6X*Q2P7k0S2=OSU|G_2{6ac%qpt)f%EJW+g zY#l5ai>gmTbS|e2o@2l0NO&&=hz&0|LT~llKy;}gY+i9|ygmoHmv)GS3HpRf`t`>_ zV=4@y^Lx8NHC24wT!>)zrWX%XuV+yt!w!iGVX9oyty z9i&VVFLSHPE}4WS*XOz247{gx~;Yy?Nnp;(yZk#i zfvXaXsDddMX*xxR(e-a0If{|=alALVjT78`K*tO^I)NXHmVEeE8DHN*2gy>`f(3mt zBk-!@h_xfhpC^#Ki5@T-Y5phCu&Ti6eo;g?1f#~)5gY#QEtmSl`5JRtIe&rkJ3t}; z9P3B^$LAGKm^rXj^9^Pg4k7AB;HRE}nHW1gLBn$SSFVUhRr0Am2RJdrKYz4|6&pt# zO2X71RRwD6A6)~+IHL~DnZ-*R=gSdkzhpkfaV0d9Ecn2DqelXm#De!@CRW9^T%qt_ zcvPTWa_8)21E?I(e2nD?r8=i2V`G0|izdX8cE0;}Gck*B(m1~O%Z@$mdCn(vVgU$> z-owB1Q!Y}2{Pb6ye*Ndj;ziG0ga-@6AxIAtzM=yY`t70?B`uNcfMC#6$d6IfBx=L|(AU76YR8m0YKwnHkE6eOU=XUS4qe3Z- z;|dZOuwL{8ZhuR%1H#N8l`8mm-PlYGQKe6Onl{ZYyayCJ5h-&a)0x>z+v@lS2BirQ zJVsqtMc4>h6pBcwa@_v>>L(C(5r=QW(4dQ~bQsnF_$!WRdLCalTjsmJ-rng-`*6TU zLGer$)>hp8BJT=u0z##g^=2#7BXo`22euT*f{P~qJjN1AHx29jM@>>?P}9W2hcv?n zTR)kUan6_rD`SR{4N!i{FFt*abA372WcQLMzaETfWG|U+cr&aoPMe zJZPLgpkZN=CLhH(od4(N8F}5Fmlxa9!yz6p$n%&8IPwd%a(SbiEMAo#@XICUBn6tA zVBdD4)?HQXj|J1IC@I^Q;_mr)@n(DZcrd~kQo~1#pG>E@n{uuof)(wJ`)cfXaU%N^Xy`t%AA5SFHWQ`hMkKJAFX-0J@W zI$CR_+R;9e8*~2zz*LY7B32F9x_<1j4Q%jh?A=X>E@^`cyz%5XheFh6v+%x_{D@Ws zCkUo&Z^c@zlbD)kh|bfjXD=;w8FK>Lwp)pZ!eDr3ra2%Io5EA7&zqnd7b1RrjZ5#> zLun0?+-oPVS^J`S?IZlELiN^k;R^+^osF%FWeEa!>Fk(;qAr$ZQgp{M~XMMi~$ z85R%lkzZR#$}CxGhA&*=0~UO#bCLuJB8gn!5;RbZzB)W;m-@yHtMiQvtBv55)7~@S z0kbq7$nDaxtIUSrL5b2$R)pO(v{nzI5;W3{dDr#jCwM-ceQ%a%%B3798oy8B+dB;x zAEU+;5FrylK2A!fR^L#Bc0=B4ey#jpZzudIO*w1pcHev8DGwHPsD%5xxYlGfhG);f zA~^_XUESNv2N=tMyhodhcb+*d;?!NoeP-6K1FQH0n zDr~x_%YbTMlJC<}#(hWOse$82W%~Ni_q1VYOfUgK_oEmO8ZOsvq$y;t-F7R_u&Nkb zv&EM;>U>@DP~75z1StQYS9-5Idkdp(Rbtkl(^ds7oV*90T!Q;=fDg|!kXyC;fyVsq zmv!;iYJ`|zETUgJ(`gS#0?hF3^K4b(i-+t|Vv$X3*e+FgWB4!&V2fP&iX1YX%VnXT zEAS4qHNce4sONF#EF}y2po#zHWcIh$33;1EY`f=O#1~bUs82HDNfG=62Q;*4M{QRF zUUhKG{1;pWPLe6daXt}o8W#V6Z@Jp!^1`-rN&4}Qlgt5$H+>-yP;{DT6cjFHm!Nn* zT%l3wO!o8?xGS6;wQ&HB&Gl!r>xGG;=`Jdc+Ci(23v}n=t@R%t^s1kObAe&>cWg2r z3D4sUw4lIw5(Q$U9{!C1EhT0WS*2d+`eW<$KS!FZ5k4M?`C)xsDkbj3=UnZ~EUonG zapLZF_6-Kt+J9`dM$-tTSX_87AiVDZ{fC35Tlqx@5V}mXqu! z^YAn$yzUmPJMK*dvQpxZc^b3?g_+lnozWOXIfLKUch8K1jZ@;vC%zXL2-##UPBb+Y zn%3z{(6Ha2aJd`^<>6E8cH|$zs$SmA(a1B_EjBUK8ex~r_FNtBG#p}FKXu2|jhg!J zMKDK1EL%{+VBd8P1SJ*pfm(*v5XmfHM7j4LTiJX(BOCet2CVU^L#Fa2vg6xyxM)l% zQ7KE)4&%dZ@>@_lIL#=kCG~oAGXt8?A_0fK#%JFi#3L3@1ne!p?UY62rLyG~1YHsr z4-DOhZt_5SJnzku5&mlyJ^wgiid?UYA!+^Xg^d_@)!D4e=3j%3pTJsBHcJgsdz?(D?vt?7 zhmbH3ZY&>D4rSMuP%1w=70)I|gzvl!6q_2`bX(GQ^~KCoagtR-m<%wGmo#CtA__TS|N}j*tWmS*12_r=u#4HCdd!7Yg_M+eQC{xL!mMph!*oSqQ;(?}|IU11-PJK1Yg^jqL@f|!+nZu4?EUoy_}@Ph&HXtfsvarQ442)|Z zhk07t&`bN);1H)dl8WCGiCko(7}h7#8_rgxxrc98Y@a%Z z=N@+ZA6&B{GmfvER?+tVffkY|iv}_LOzMGD5ux#h0x4!5uwxvNTq^NeHbi&B{w!~a z)hmXcrV^K}6uagoE&_UBj$CGt{rew|)8<~Y!32TVkyGjj#UWsJCiIu?POf<`e=E!E zstZM6A=JQ>8(X9hP77H>X=%i#S?uktvfGQa5q4oj(%!$b7B508$y&EI7Qsk8Ss25v za8hDgNcxq_^O}bne34B~KYQQc=M}gku-$pZ%#@JnoY)&7hGfGr;4y(O4Bm*#Tn`i7 zrMoekFBG1L(TAkVs1ad(5vQKx)R|8?C<7G z+L~ufHd*1_8gfO{q<}xvFJ}bIN-bm){Kqy2@u7BAe(dkmf4DvA4Ig2%ek)k&HzhKG z-{3h6@KX_uFa)1LTeg5W4_NFThJfT^B`W6d9f*8qM@1Tf!ar>R_8gM)e^6`2KONXe z%ocN3|2@71rnV;~J_-To`80JUY+u1t3{RLm$#9+?a1$lp1e@L7*X->M1;u>EIE7vV z3bZD&>2S9I;8qC?gWE%qU>J@r*WJ#68v97O24w!!w4v@n=OzJ2j*W85>xc>REKmRK zwRDie<5MXvn_!6vBvUeWz@7)DlOhZJu-A*_+o@}u*!V_h(Q7MC;ME8i*sGybqCfk{ zl%W=kH&m`Tj|<=psH!~Qo!^-HFg(HUS2ZN1Wyrk{*-Kv^#>;8J)|aEnvSAp+_Dw&A zq6mtX)qKay5#NFvMQ5qK)S(LlY_Req74YX#d(o%S&IH!q8V@!;)x~-cGs`E8zrevLHidc|4_qQLn8PS zJMV)|jz*D8ynvz3*WBOEe2h}ZSHdmU10c}(z5I3}zX?B7=IYK*xl*{Ou)-kklnK6b zRx*<{0~N~!->=)lGo&DH+_=Bj#!xH%TIJSA#`n~QwJhMk&zEOr*WFZfR%sxo{6SB@+*e3-=pNBjc z!N=1M&#$fprqXF(?w9@d+nX5N^61-aD3kD=p{pNoM_Q3mAH?Jf0Dr8#nact2Ok5$D-$4^cuR5s;SX69=Yy-ZkV(zSrb_t7lo zo_u(NS6a;i3hF^d9=5X9IJmZhXUfORx{U~gp`i-1j_{uBCnN`b=Tl(Xuh6#`bM2u! zwYHDHv=TbdH(SQGrw`HhC%dZX`40GW5{z(R>;>M~i*bE&9hssy32oFkK0Uq-iRd^j ze1&J}Hgp6AE)4X%`PYN;k|ubA-Q3>_llprR35O&R_AkD<`V)k$|CNU%L#C+4ommD* z61Nz{?sG;qtsqyemf`x0L8&NmS_l=+PI!`vOyD+ta_)y?on$FdB+QcY=B%Q}Pc>TO zz_)atGf8;dRl<`3e4vFtZFKO(R+i1FTc=la41IfnX<#T zmIDtd3h7+q#-7`RC^$EF8>F*1ZNRntH914@1p2la##gCUk87l|JP;!G8Lq2kC>4J~ zAY>3X6=f zQG&79sz$ea%!4_txofDHl@gHt_cwnT*2zw96}U{bl*0`j2`M~5^?fA+H=Qc; zq)SZRvAa;G$H=&#fq8I?=KL`MbaG+Wf);aVoXGQHeC;HLHw12+Do)OHC3-Bp`W!%8 zqZK~Ru1s3=iX}5b2b>>)oZ$v=+YHFm0q33BVnl3I3~L|c@y%s)F^57SF8bfq4T?2p zJO=gpd`Q_9CK%{?^x1Qw(a$4O-cIi%ralr8aUVL~1{;s?jxHDgiQuFw!}XqD+_{3r z)w;~xcy)13J0q7{yY6h&(+9sHB$h3r$#VT7wY^lF-t!j2dXb>s&pk-t8F;x1cp=I; znTf(POqEmfA?@cZn9eSaD_Q^v2@((Y(vW?So?SdYT0C_D56EnNCFa*cD{FXlvb0`W zg4$=0PX1wfgEbgsF^~*VOX~!aXa$@4#n(l#$wi=EMBs0vFPo0Mbr$kEUw$^LPa)ai zd1o2efSO#`9ZBgDFg$eK^W1w#yfhwBlKu0@;h`B|znQMm_$+0jEO1{i?2C7Ozr?2Z zT%QE$=n*mwcX9GQBtiq?30HE$%-D+yR-F*m&OrMG;FigTGbGl4N0c-+d2Q|eqeHmT z$He`ib}9FdDH8T*e7;_R9kHnQZ|kaa8B)sNR$+T7Wa#HX>lEyIK*AaWg1Deb7J{C? z!x!8vR@h3t)^E?`hJ0wpm-<4l3?y4!O84=q(4#<~Byby~>AI{OdFqiNjNSNrkT-k> z*CMO|WUe4zT%2eMFduak{R!b%m(iaQLzO$U-s_C0=8DhY`|HZPLZyKV>^YU*BW2t( z4C^9|O>${U)-y1C!QN9UY(~|(;V}~*qNyRtPp>WpNLASXI*ZvR4FsPe4|6Hw{HLHw zS>-wvHUg&OCSg+HtQ8Et1Q@U{zk2K)?f;3s;>V+7+AO?}Pka8vX81zK0WSw&hwq0X zq%Y?3L=rv>_ucqv@WbK4`}-v3K+1fd$pAjH?gbA|bzD{I(ta0?Yw+0`a1$<1dSL@t z#P_?ia84RlZV%1+`&6$FcoDHx_CqSr7OJ}1S_ToUGMECWg;ZY7$s&|i2qMHReiGjV zbSw>ZMBe9Od<;NH0WhO!J8j*oTGPvlNur@w)1ExF^ z&|qg6DZi`5TWhOx{|F+tyIIfvg#d;-*HbS~hfBh$)Vs1rGVIfwEB-2>1cvp;<2LhE z-&M)Hp<6CPbOGe1VRtQnf606!$C4L7mmc3#57hpZ87&?6>Z-GygjS=kcXNFpG$EMz z8p05E+drJiQEhmC%$_OR)fXISFV`4&YhSjPWFDW?!$|teuMZT3*UPLaS$VHtX~C#WZ{vrjBcUXI$S9N{M4Xv*-l&T8~rgrE}!C+H@|> z6cf0xy6kceTz8FzWb=Bj=3N}gdtdWF-N_dMi$5ouALWx!_lE_5Bg}W1m4?JU>%UV| zo6OO|07-j$IU5QJT)5OrrS8VVOdf~<<<%l6MzoTGZn6$BJbMa6Nn5%k|; zK&|nH?t>aCxKxw-_-XvSfS)B>yMZ$|)}6!p*L6p8V_B7e0iSgD-VZqeg89KFk>A{*)w)gk}Z29yC@2!jY^ayO1303b|HI`C0WW=i4c`W%9d!c zWGSOWSwgZ!`rVJt_jg_WbFOnvHRFBn=W~CyryS=McEYVs* zG&voPC1vAmRKOCTgcZD;AgYy@R~Z_#wG+Nhf*1{U{Z{byP6-zuds?>&r#$brW4h>& z2E;_J3LJhOQFZi3SMj=Al54X!*l6#LA30EBYaS#9K*ST@k)u}{%A7cbozNK-o({O| z1u(?6?4S*Ze-oaKCqw=*r)mKBdMLN9pqJ2$d>U_k6QZD}{N$wE?tn(E0@w!rNJt8f zUz7U13daOjm@%h>k#9rhb+mKp;1l(^I(E9yR0(;l|LJBq#u%<3Fr!o`il6*G4AY_j zDA3TZo6#4HSaOw))*a-UO9MHN0%+{v)e_*HtT(@Tw)C0(F<}Wn%&VIrYu~$1x{R44 z?}>)|eOhYgl1{Eqe`F=Mh30w!7KU{cJ*Zxl|7tk?QaT+F-6_aG*6UENwkr~W%j;=a zilrfgAv}6xX(vy>Js4yVVsX*hq0pXCU>qWnua4kmXVN7^(4;PfB06@2hH}`Bik2VAl_X%UL?x2K8&cG!0~uGd8goY zV=!^`$Q}S+lR4PT(gaXqN*Au;j1}jjT%cAmOBh|$E)|$;ZdrBtrpy8;^>ATmHAmgv zoBI182KP#7{rrliz?xj;dx(jN3FeHVzS=&nb!hPfasVEU4qjkOQrO=K%u5xM1uU@_ zuPGiZ&jkh*eG8=G*z>C1nqr{bRL7dR=75g)_#{{)Y$lY<&YgAVZ!~Z5h^PZH0*XMF zJca-z%kItD0?KZc2-o;AsR<_0O&3l*oBaQ`wwDL|Kog%-{62^i=!zmHLjbH_L9DAg z5yWZ*Fig!v4g16yNGqRg;>gesuAA(y+WlkbnfQy+BR;~C+5qGzyzi9 zt$1cH&YQP9IACv<=7aO1l#-(bvqW~9fj_u1P=AA+DNE!KkkZ6F>g&+1L!b-U@J#|G zf4XzKV`K32Iwx!Ax@QNkrZUL)yTG|2135*T*d29q>uab7?&hVRF&1-Ye|!bxm+ca( zeDhWCU&(S1{g8|=d;yu3!DO-Tv7e`%=_X%Ol<2=F^#;<=LOU-%sW#|ZiFm(>1JzS% z-~f!-%{dCzwj-vOcCx}DXJ?2K;G`75nrvA&JejYW&yBv`|22+571uiNTzhf81jR-d z)Xt_*p%y*KGxnA1jbVQYX(@)qZ-R&Ryil=yW+3vbAl%W6HEa*OrlG0WU zuIG>W&*bq0I^_hoZ@c|&jF4;Z4RE2%w`wLZ(m?R45;Cw&ODY4AZ=klb%^ngR$ulLd zc!!Tlo%H%>j48xkP&b<`qpIbg{}3GgdbX=d$onuLFPb2t;ZdY#B0~(&rh#$34RE?x zUCgqqWI))s?Jw9s)VU3Wlv1_(D^A7UxC1_Thbscxe!S7&AkE+ePT9EM)u^^{85`u>~2=!}*GDZwVc$ zVc`@my71QJzVjAql?TtL zH2cOa-kLYrP!kw(FgxJ4HaX%`azAYoePFZgG3Wk8jYN zOeRy{X>##r?7Ga%B^S^)N;Jyt zS^?H6;}CFxMwk{>@k?ZgIlN!s;(Ccn19VvKd;?~)RISsGHEWU_v@y%Z<{FF117LIx zb+&`%@0$sV=vKUS3A@ZS1{qj07WW#S@0qez!fElGRm1BY85CSSN~#ToBxvp0CPOvT zjD0I$qYhc2^@bg=YdpDNG+7M^7w;0jy=s{J(GLLcFMPCPw<$Vm2_}7D@NL+&>Uf&3 zY}ifOdi4N&6!h`Dc)d6N$pqoJ74mdmTsQJ1Y06|}P?QX`j~vyr9o)pCW)ys@i!`ox zXxux>pf`XWd2YZQM0l6{WzONYrH})dQ0l;vT6=!^=P6H}jjktGIWr*TjtIW-xlLL9 zPnD&m?Cv?7A89#m51WJTH9CB7JkMMajq{Fs>cNUMn8>?f@rby}S#A4hUE8U*BIgZn z=pCwwYI&PP_2;EZ+y^#tAMn(-ZGLssyt(_-qyS;5?wuZujeN0sW({$|N;%Rq^m+(o z8;9tJ_&BJ`^$w6VRlmnTxk1kGSNb)R_lo;?s|F=aPKMxE5!4c^eNYH|G4W)o*N z>biqUy*KT640z_3l^4zFQiMS`>a-M`)=n#4pE$9}285EsRs9eis2l;*+;ml}^qAqk zPX3_;Sh!!oOr7c<0t!2~i#Y=Sghl~BlDAqw zJhH^|D0rp1Z53_zsJX}DsnF%8!`bHa6^O8lXXMXVp5hU#m>E3U7QF!2e+MtMVZK>B zD0!qFjIcL zov&O}2G{q3AzA1E@_P16bX|lE;KvsH0{hR6L#ePad+iRe(5%Ufql$m3SnSYAB5%8h+Rn5&@TwOW`CvC#J zpkoPrQs@MkIg69$3vb^H1K_+MLDb8-Sq)wruI3_WiU(7YiE>$9{e1@jh;k^Oa9h>X z?Xc0~$A=i<;o5!CSEHmV?jivB-P0G^;-Ff7{u)T1E%4?~)kQrcLVr;*Oqss#wb1U1 zSko26n*Le5F7?X>XO;=cKQ(bNv8b>2@lF)VZ=WvDiVc@~wjW2qBJg;DTgLYZxNPs3 z9D=+h?F;H%_~SGpDoG3wz;5YrX-RCj^&XJ+fz97YNhbd&dT$tdFJ4x@Mer@&zSQ0d zlssrlk|Bl*(4aJ-4V30C*GlB2^qV^5c_8uLDwGwJZ!)~C*iO{;oBX!1IdZywWsmiA z4cNqbf}RU$?a9(SKhd+*#OhG7cjJ2k`S3=!KPEA}#@+2E8Cwqe7Sux0igxt$;$_$A8NN9mWWdb3-ZMcBjFQ<37 z*RAxfbiA#*32#5xWggysWo+eS6g0d7KYw^$x9kPbvTs6QbPbY~MzF1)u`dlaVoe8} zZfQa1-*)VXvbgcX`!aX6Ejr;(%?GsPvgDO_J`I@7s}8YE zh7KHPd5TB=jqZ0QJFNvi*(Yj|Y+#95KOX*~sT2YgqfZ4$bm!uW^W*(Fl6jGzLtvv_ z4}yWY-{9c}j@xE?A&01NKN8D5fh6HVHsiaFyW3nX-2fjHh>5s;B)bW2{*fqq&SVj7U$@$volTo*GR{TRbE$ zuM7@_Ea>R?q~Dp;*Ll36bzeZo$JKB2)8b^Mj!&o7&9G!y0E2A5XML#BBpV#4BxwLq z73KIRD#c0HO1*l2htY5_)ra=0)Jf5^gzX_-Mu6F)so+M zt||xB*7lI^pHW;ItPA|Mt9J$FYrd@zb z5ZAx-ttSdN+ub0W4E8_Tg9wbf1 zBH8d1VB~5Z;0X4UzL^2T?kz`M$pJ@0?GAm1uoGNvYv3{Ihu=$@l!mvy!djyHXy@Or z>5hYY;;B~I$=ss$1ctFz`dh<1l!}9bQz^;CG0? z&+LdI2jZVhgZ(jE$bi4a+LR<)W|^V#4swYJ3ZZUtYaqeE+$HV9%VWj%rDuk5+TvV< zPHP53AE8wAu9sGuvY*i|+DY|zIj@apJ%bW5uFYf9!1cUrVXI13!^xIS9pOHL1FOe1 zv^ic_JK{OS$bwW;_lLrij&H}AJHIH~Jgs*@-!d^7wl(PLDYysyBquq%fL zyOfT}vH@DALM!C0qBGOP7Em~Uf~~T+-B`0Y>+`CyBRju5!^X-TR@u7Y@_2k+D)P`c zgBh~|m}&ECTQW+f2Ou)~z2|NwG#2bdRc8Rs?#%75xX-rN1bxH~ISOm}+0PIBwY=DzCOv(0g@aOP4-3{FrO$Z%ZqIA7LLKeiIstk&V+8QsKUBm3dpAuP9*= z{DJ{xNn_syJ=XCQ?+hnav&6do`R@S^hreL6Umh*cg}|sLw(FO%Yga`m)sOZNSbJt7y za#lB_ts}xX)Go6!I7~9)p>b=UdE2uC&#uv5+Ox&%pm99Jd<*y@c1mRNO%nuvs zcyJm^T)%FxG4y!itpPyLzcZuZ@O<0;F9eU>Ns5M8!|$B4-6d(sYep`ySKdD{6 z8s{0CnZoEkoqBR82i$iIt(e)73SFR!+Y1uuk1?~58L0hjV}jg;==AtJJnShEY>jDQ!g4V1t?pXB`DkG8@LB_L-^wFtzZ zP@QYZWMI)BNVr&~J$dt^-Gp~W1>5!F?t%L``@91nV2TQts#WQmyP;66#t*+G1U%E@ zf!&^lm@?;A*R0@%h|MXSNACe<>^U^M;cfa8BYT8KTEWeDTSFp050rToZwAFY zT`Bvle>F3K1}|rcnX9wlLC%yNEqIe~wRiwe;{=9IAdHv_*UnAz-0v>Ho5W{nIqaqG z(th2hS14!wTNj0Ftm2`Xvv=;;TIcSW!%kJO1(s4Iic0>QKxPzEDiTY_+o&=zj>6DR4*RMF&Sr!^ltF4gp+YBq@c`+`&BKS)P z=#t*UgO!(qQPE2(bW+#UMBX9KQEjyShx21C$EDnP_sBM)%3DatR&=*%R0GkTcLg&Z z2jo+qUmYATtw6+jPMxOsU97*fY+Ii07lk@%g?L(=@>Rl|-iv4ant~x%lE2XJe%~BQ zWzcz3ufH{6FAi&t;dQnj+r|>wy2DUqGnut%k0S1RTxewt7=qoz`XZvQ-kwVrC-KgR zV+S+8ad0`o(+;0Wg$C?`1!Q>!vz{>?z+y9RDbVJj>E}vXnyl=6C}!|Rrpy0={Uh&T zs{h)FBLsP4F*`g-r2525wwO(Nbr0GO@J93E2qtu&DOevp#=?a0U48&pOvmoWbBX*r zCCc#MFx9-n+Cii2&*#FakPZ+mqJyrr zk2in~C3+RK96l-mF0XF=F}`q8glKe)4(^as@WS^u+TK5Ql=;Sl2A7z&ZLg@BCzzaV zgr1H5R6+7~8=+@&CjK>Di_{ENpPtT`FU#F-@~#mf@}9>3v~x1el_!^J$sznvHeS>O zh3xW5|Azm*m?L_WO9c0r&j1H*lQ3IYBop6iBNsFHSK#rG2{mPpuQ7P;0pl|nx~gih z`AS>dFcTcUWlBdLRh*rJ#8-zElc+6^!CAGsKcxd)Zb7p4Pc`X=f8qLC^hZuz0NfcWi3h5(^ zxQ7-lV9$8dK{T-&3fQy`@o98P3ahu%(QOOY@Tk=@wCG6ot&PfC1J-liaE!}FKRNq0 zFCzJg@!j`O{ez#*mC~w!4!2`4LAHWl8G)S_GFCmne+Ldn2s||1G13kDn_|F+JIt)u z)oh9^J4;{se%m&%))l~0 z^2BR;V|S&rTfyn+%EiA^;jmrC}37HIapFn3z65 zYG=5IKwdP@P$`5qs6*x{QYQm|5=Nn~skMf8(PNEJ&|T!}Id`4(C}1(A>aYs=(^Yi4Xp5V z5VxxjdIgd*nVZY+SY}a#rB*C(dn7Tnj+VKSe1aVe2xMv(2TktgC(-}Ni?TLN7cZy6 zq6nhlmFt<0EO7>aXBTt$dC1K1)22nsWSvb{)LSYFlBnAVPv(0LhprqiZ72rgp!tEs zvOt}R2%hWI_FvWy0EL&5IaZH;_TrC>b4ty8+kJv_gOcc-X(Gru6l*Dcc#GbZIjAl} z_f@wm3Hx`61G=@$n7KH`^#{L9UzrYSRYg=P6l6*QKxdjQdC)1bmjU|(u1_FegHzNV zfK-~c@3eM=hHo~vX-1_uJhGU^B_p#9*vy_ z%+vPJX!I2DR1fnTtHEYy4p5|OpwA5VNfIMd-0K)}G)Qxh^~>e!3nBPiTto;+WwxBw zjORP?XWp3658!hTvD}%4SX^Pj&AVf***Z0-xkIGv_{a2#{~ib@S<>q#&rgBbiB84{ zeFKi7RQR^{SrlH<|GVm=9m4eGr*TnwG$F#{ZU?Zxx7=M0YZA8{dvb7M_2qFWAu}KI z(hSr-<=>^b0?zuXj}1oG<-Y4oiF`2ohw^1{)}?P9fC2TW+TgqLARdoPqr=x~=QIju z=2rkzX$Y0W$y3#6z>vs(c1X=54nqT&(E_R9`@wRy0-&AT`=|k0lV?zo#whekG7p?w zZK((Px1x=1MADDPKe5924%x^uWh_|7Q$9K#M5OW`UN_(TaJDu#q@4`h8Jqn#^>7EpHyJWz&E!BA{?g1RP@cUn~a)LivfqokVxKM4s$nG+S z_XkIy@j#3Q(F}7q&+sZ0k;l)A&kGN8+(yqd`CBgh8w>4)r;DHXx+c0EAg%QAh7A|1 zLlm@{_`7p|-OzHp@vVEy&PJ@R|BoXLLmDcoXF%h}$$yC?b5!PutZDFMt)a4-QAv0- z^VjO12A|!Ounxz_Y6NSXw%R>aQ>aZQG4P4lC%^%z-wHRE=^{T@bn8-btkY{5 zI|Fp8hb}ahZn+t}kWrliuJ9BX+wL_F-Ui~~Z60IFD_aj@f?8OelGtn7r4G;x`38RC z!D(7}`S7`Cq2?D{%8hb1CoptzJI)STU)i2h5zO;s0EqT6-z+`0gNPGMhNH9G{Ul&Q}`w3--sw}iZ zA;wnZ+&$~oaZKeCo2z7nE;RHg!2Ix!?N?tIE}bKg^TZqYfr{($swL5p4KP z%%&EQDin*v55qj4!djc6b=GYF?M(1S@>&W~u4Qh#jwg36jb?h4M4gMnyelj!yq!s5 zZ|AA#l_pi;;U`fjTLoibtj&$Wtz3LXcn}dNkRHN8Ar`;$g^asr-rq*%?swl^BA%uwrd3( z0Ps??4d}4vA8h!W2sCcK@0JwK-Mk=%IOj3s#8XA9y#`{^&x>OZM`!lzpP_ zmbG7j;MWD%zs8(dG~jof+YlWx4*0vb_&IV|U$eN1!Avs1d@Y0&a+&YiuA>~!wlA6f zhtJ@U&t>``!XL=NJ0pd4z9Y29u_X;{sq>@YMVmQ!KfsNba0~}qP72iXo?#VKwBNFU zdh270i76j=KO>QOv06V*Lv36aLQfrtx1r$n%FO`yKI2d2SKF#d6y16G2$ZXfW`%tz zGGh?2`0`x+lg}>yd8lixeFCNgYI`)Go`toyYUM!^KF0)8Q4Y?i1M04ssB&5U+d`C7 zt49&vmBqV|kIOUua^Yr%)xmVffMI6TUpH05F`sfqnk0m@8r#B1lJZq8=Ip1ynuU@_ zfF&E(sQfn^;BB#@`iBpjH0?;FRuUKzo!-B11R<1;%0SUm01;ub^107O>YH*Vhqj`cz0b}hUww?_!d9UWmnCcPKJ#v|3Ii2$%&I;M*niyoKD1p$7LIw{Ai~f zs0*89_tq$}pqy6qzO??3r@rEYU)X~WeR zZ0Qj}Kh1!d+>42g;pEpD__f~?+lt|ClN;sgi{-$tl|&Z*sJpTF1zOgq*_d%Ymsi(b zS7R%Rzy5?<{EFsHk^a(g*J6abpBk0IM>IKkfmC%^e{bs4Pb8s#j!(obLFImVf!LIry1 z-W;$gbyaz6Qpjr-QFkjCXW|xArq9z|KSjM-lcSosuf_+1SMpv5<(T+Tev`VNCj$p| zgX;u_7a`@aa@TvqqpFav6jCuFSsJn{p|r$9&TkMa3>Ch1VXglS+=oDbI`odwRpUh{ zPM3#T5GQ*l%+cL<{O7?G%Zu+g$oVV-9)TK-Qs32{cgv;5e7Bl_HyHR7<;1Pa$2b;g zg5!}WhsIO8jbz{~?$vzpiCBIDn7m5Ta5ciRW#mb_d|Gu`N{SL|6Gw{;o5Oo&^eJW1Bl|<&rwxy3x0xOZ|g&2@g zqH?Dx;~GVh+G>>z2xfn8y9O^o5JzAbMizEhyKgA;KaKW?7D*tR3;g^C{WZ=G9(?Qx zlwN4A@C`5G$m`4le+pb$UWyfdTbTn7I6E=*N0#Io-Fh>HI+yBHUUZF!0)4fG3O zsyuc|u|ijg2o5?!7Gd6pC`|Mh0?mREdW_;r=9sH5fx_?GvmFYCkrd!L<>AeR)eMGw z;{Q+&Izdph~gQ3$Zrf|Y>YkT8ApKVAU~#Tojxu8DH5 zfdM_sH(*9~BGBI+26ey2!*Nh9_n6dlGt;Qb@2Y%j?$8sYA7T=VY+r>g-7aG(cGcav z3Mo%FKHTlZGi%@jWwXG%y|gzM7K2a~OY6@n91Xp!T%y+MN8&Z=lj$(-yf(3 z=joqHQVxwr^Rvbi*1Yxkfc4^WVkxgN`>83-FvTUtYX$#)n7eQY0ANIi zw{|hd@JL09gz5(Ve6u`Z60S(I!1xN?8aW}TKVThjYp$34{J>s3>L4SmhBKW5cCBqW zHu%}^amA^_NZ15A;~I#zsD{vPb1n$TS34ZHZk^r%R?*W z)F1*R&sEy8j^^d>tDM|6KX(5E|Hp9c7_M#Gf2=23Y|JoyAB(;_8&O5>M3%{_@H7GE%&bIcWN43{?NVi-(|c5{!b{e`P(LKOl%@fu9+u zfiperJI>*V<;A>eIBan=%;g~{bWsA=+uKk1dQer~eHuJ=cQ`~eEW6Kggr9qmuqH#@ zGK(%<7$9m!%^m?7O#(br-)!QW>UQ=%HDv(F3;ir4#E1NL)mP~U|Ml^+pH-klim~Y% zU}5l!Zlqm$++Dx++ub+8*>h~2euMWmAQM)KPqJ-aQ1?DHok*tz06;a6odN2pOL3E> zZ^fHq`s&;Q=D0-838-^Q4p0?mOFjXiT9qBer!2R8KK=lV{nefxGdpQ_tQfcacdjkv z(|JH;L(d5b1^Rxo%CSJHlO-%hkhu3r8%2#W`xf8K3*DRmw&A-5>{So(q&Msl_n;v4 zLv=4)#)7nBch|5GBIU+oB)>oqx8~}Fi5{!p3sVz4O&fU2c|QQbwJ(a&M>ea^_pk)g>47!WwB{g)F=FJMP#spl()caw`u)8o+oDH%d1Pt+Z^>9@L1UbP`M1p?h7pc z{%KtlMTkj`&8@XO z3F;eAi~&s|5O@-jYmDsiK-198O~$-tjtxO8u5;G8n|fZKbbAwL#3G#ZvfnP3=0Dfa zyqNfQac0<1gd^2GG&HW`GTb$G_vFEvZ3B5%G4~w6mOHrTI`W45|HwKla=?skfpPc@ zG*Kd2GJu|xLSn(<)27ST=TFkh4Z_x|kSI7C1F8_+#vo&i49WvV&!USTl^-R44jbGe z8iR1}UlL{N12{AkS-s$sduCJY7h|{8%bPqc2EwWp#=|xy77mnt7+8R>o-heiG&Dm< zfU0@UHx~fQrmvDmb{lETK_0VeJfV*C_<8tR+5>&a$Ls?aYhEF!0p9Y<0yP=XD)ubS! z!q?T8EY8>oXsmRGqleG#y+GAlu+K^WT)#7Sfb|vDQ-0#a)*8IAoaXWw>0Bx0%c}u} zznRrBcy@GGMD<~Szu#A@Y>xlu%D#^dU-##_`b4AcSmzI9wq^K>o^>^mRR%Ci&%j%M zSsw6u*6sHL8qj=OP2K&SPNqU3^c$cr(hNg@_uC}1N8$^oxtAsWjTt$WNcP_4+Hxw4r1G!mq$N zF>J!pc@=1-{RZT9SO#Z4{Ia1?dEeMzGVYLRMbhQqD)Y4B=@PF!|CQ}jZvM{`0mDb4 zRjxwmhQQSqhME}^aLt{9lc8EE&*TlPwS^FHgDK^$fufv>L}mOJ)ML4E+p#Us?^sc( z^ZHW~HGx1*;i{S|VPbukmzT^Qe~q|MBYe=K0;bp^{;IHGGN$oWn?472HA zhbn|@pG7e97VF16yz^y?BI7qXI=`6K$`sMb(eK?71EF&z8yFd^5d$bAl;%S96N4Ay zgMLF3z~icOG7pK2#gjm7;p{M)mwkQd=$1P$Je!{m`~2r~P#r;h# z&2(%U%67iOEFJw=J4wP7$sB~^DV>+80xBjqKc9ArIwI~THBQXE<^uohA=f0`2R~fE zD+CVQqZ+l{&XT7+;U)=OjA2(peiC&*ff5Qg!i+o&0 zJBjUXH5`s8Y%CK{B^;_I@wH5~=tVA`Z>u+v9I$B|hz-4iQ9~EP2MNo<#W{n}xW5h*wHq#?tAfCl-oNMwb&0696&CmdD~M$U ze&ml~E({ZNQr4v8bJ+sNfahVT@e+)2iL|D01-G*r;7tB=yTLq~L*WqA!#s_-x9J2> zX3Sjb;J~0E`uMiWGHP@KS+L2@6)1*O*tpm#`p?|5!gL<<{;}(JE%@M(Cw^-p7}|&h zM>&IFT-qIFK-$p^j2Lk)^(5!uJsPhOcNl`icZK_nN{vfZ>T&)(nQNs6>T*mOSt* zw-_>XE?tgIyf*9s5UxPi1|ki!IvqHn1APQ<1;6j_jE`OXE|cb4_%S2i#sbsl1uZ#m zbNH}gE#K}WQRI0kvYhpmj5z2Y31#y$$X3JOdy7lKjQ<_ya8rA7Pg}(!)(vNUpkc8c z>AG&hz5G3Rk=LDs6VCD7af+`XKxMEn`flZxNF|eAk%@2Y_Y9 z7b;t~#SF3Dny+P?MlTLM21GKy&vwUenvs+7%CnKIcACMk^p~ly>uDuR1}fBli~?G* zOw}ZDya<0l#O-Ad_-J+HoqtNjzVv`FbSm^U9fkwx^x4r0aRTLc-+Nz*ACGG6s<%ym zg((TiY4nC&>`?o=r1olOrfI~qfiQ6?1*3m+=_%AfJ$-$y7n&6M%8!TbeBA7#lZp&@ zaE({S&u?#~YBot$b}F!XOk$e1<3th2JJQg*y`% zt=M)Br;uYC{@xTx$pGD>+858AIsJ4o=WaHq&2CID)`2MzuiDG~!sr5!c6rsRx+3U_ zT8hTlx+u?TLDuE#X+sv#El@$Qugc9%JUcB)hE~o)ADe^2r29u)P!XIW>}`m&x6Fm> zeiTj)4!UfCoAY{`@&?>dS3S)9vz|wYmvHjuFmjtzF29Z(s)O}AGxL$?3B1Ww(aqp~ z+|b_5&}qz);o`UcbqJ!?jEqE}P1Yy>9x~lgTJ8w*0uAk4YWNu^8gWbMcBT&&(ieJ? z{!DCk{~E@vp9t$hI2T7$0>AID_g$1{DGFwbD*rJp)TmsrSxjHrZ=F~kB*1M`2fJ;v z!NrUEf=(GrNXdf7_Uy{J2>hx6rbHxhxXS?wL6)_;X+)YvswDM^YK5QgsQJrNAP-wO z>|Wed%`8qsraCYNN`l~ZUPRPl8ZLZ$e>Ywc9!r&3U}dhsf%ghNqi3f-5}x3&QGb~0 z{qWxhx+)9an*43tI$*&dwc*J_!Qgu8dm2=832Y7Mk^{6T%7BaPR-8;hbB^KxB|Gs) z(4u)(G%=V_d|S+U!aDuh#(6wxL;cBSuCM&tE54LH{Q!9la#21o5IBJ+-E4i(z7`(<67!Nv%H|{;Aoif8=TvLd=2GnO{Fao52!202i*&yGCz3l zuxf7Qj-gBY+ue3Ai5k3QuF}1Go?f?DxmA-upN(Ca14Bd|=V`bXs(tc;6tA`-k{!YL z(*{`UD^S~R4EY^1fN;~8qS1{!@0!10;X|yNI>E#4hGP%ujNY||MD*+i!*58JL`wW~zP=&= zYLo?Yh|{5)9ZC!nt&HQfG=BljSB6(UQw{KJ!(jM<)y75s3b%DNk#r2_-F_Tb4{n_P z8la`^Q7bD{CnN$zXeQfpru#fwPZNN#-3Y(@*)Z-We2r{?OJgqEzXckBL(vU$H`LBZ zf)CV5<0f`%_%R!9;eF4%G5O)I;D?|3nYl{|&cpH971?l7M3*(X*o45f3`DXtVD}2( z5Nl>psyI!oXkX!#+{?%+vQ8*u8G-de?kML178uUD zvpa6KQS(cK+cXL23V~v|@3!_3$0?`?y(<+_$-VFG(vi2j(lMc)63p>Vhjo1nhH+PR z!z-fuZ1=9oleD1>FaQ@hjYB}N&g3SDyz^qHow_H|&*p#h(R(z%9x6UxOQZ@AolcAt zrXSnKy|t$i;-H5Cwz#f4Mkpj6s_jR(&(uxC25__8M+fJfZ&nQ%+5VrBpw`A9=qK#W zrPEV_+fSTEY$?cfuo)M^fm!|I}XgPt? z{oVsGT^<9u+c?l(fJaFXPjW6$j7o-WjHViOp24!YUKo$A*h5y+5-+d8n-slV;CIm7 z5`nWX1>%n`3**)+_@JJ^+F_W$wVfmRC!sv@9sb8DL|&m^3?p0_cJf8(bNwW0I(Uu- zeM-1uet5S(d~0)Dgox+HcE$H-g=~KgP9wJLXKRmp5-*u}I%OZ{9R>`4%7wZ z0PFtnBqg#3nOHwInYi-Mt9HrCsC+@Nh9D2!7q=_5TR-8|v1a=DSrL89vDlNiO>A-x z-$>L#)Cr(2T_lbWUT2{r$%(iZ3Tds$unxR>McYt zV;FZQN2$E0VwqA^sDUkJkLYt6+0R@Qj-Z?A!I>$H+@XHPAHA@}7@l7a*M;;H9cfpiGMTiZ1{OC?xV$mw-FAp;k6Qzv)$O{wkQZ$stch5hL zd;R`Ju=07r$o8$!T;wahm`IRCggRN(L~cr~<%{!B_HImveiv`A1CZ~hU?T$ZoqZRM zPz2aN(-|&7)|&9ul5+};~)l6V0C-3Ulw_V?=+%eOuxQX*L0^nfa=rWIM=3cVZ8 z*AJx|{nob}tXXi)@a94sE`Xa^Qma%mhnoh)tMZguY~>gJMt zf4W{S5qv9f$3T2?vTDb|r=aPWJ6krK6t7RFOiO#$hNCA9d?t6LOkJp*}E5HG=HtzrB%Y z@OOZ=xvG_lB!HL!Ts1dl11qb2^d1plCp9u}hOYGkoS>8kpzBH-{NY+_W4y}n!8?d1 zLQK(nR0sI}K54}iL=b?~Oc{_{*XA8D=D{8X96!GeGEB!bnYf3o)*ELm_quU7a3(*6 zp|`#e=U6accS1Krqlo56OOhd~00eHL<|(TK3Hd!~tRQpQW$>3xpKcsUi3ER>Lg4n6 z0~b%|!*$6rfk17Xz4?pADnl!h#KiQ#ha!&T4s$dqt{G%gM?G)^(yrUDPz~G!vIP`U zXgH+8f@sZ9C770L4J?9}LLl`SMMHk;BjS-~myaGDgWB$XmVPk$Zor)RNW~0*JCW^X z@LlIeYic<7ogPosa4voh$jM{ghV2TDoKy`(LG3#%Y|Kw#)D~l;+Y18~+Z7ix_;u@;?ba6MTKDp}e(_184esSK-V6?hh`N0*Uy)e-z z9N7z)D`FBGDureDCHdC-jP2>YjV`tTzXZAn{Cw5927kOaCn-0-65wR%G+r}mn9$Su z_WP%14fN;E$YSxlRUUos{8|K#1sK5iba!`!Cnb_IXGt}Nhm7}&lMwDd}jp_hw<_2#MoeCO0$AAI$rBFKW#GF4k@9BDt$vuWzCD{!~6*W=ht%J++m)k ztcm^oba}Zme zxx`>Pc>vzSaH4ZZR^KRQ!8T0yx!kabm;t7DL&itKCOF%Yp<9i9*O8z+{;rTc``KCB ztlIM{H~)^TPW{d5t5yD`a5#BP`r^Z}=jC$#h(QPVoXe!?k@ZMmeRmVW?)4=}4DANt zif;*cxUWFVz1f877;wbE8zBTI-bz>(Pux?FZ21LT7n@4m_U)j~9{GFWsw)Ix30kqlr8o>S^4q!gr>uI@BMA?cyAm`ImoH<<{?88GqzJD z2=0JC+@NxAdnl%iCD2|zFSh{0;Ski)wgL#N0M5*k_`gZuTObQ=d=^-=)l(yNS1tfw z_!q>S%G?)zGYo?Pj7?BI7-7+=nm{kk{M*{al#tc?KzX5PWn#OuO_}nK_KyhJkU8+b z4uu^LkXrxv@RV}W0D2Ke{O=0U+A#~9)7Cr3K~J)^QWInC8V^t1^m-?racQSFgSs%1 z7*P*bq=JEEeW^>wqtI;Vew5@dVlR8u@$2hy;3Pwu8sp#M`FYM+Gi3WaAoz;S{oQZ7 zdha8Aw2}C57i5l-YA?WYsdEBq-Z$%DliVt(iOO&`qbg^xt#0nK*AA&F`XGqKwJ7(t z#_fPB9&n~#4g+oNdl?u8rEO1tC4-KENdZ%VRv=+=eMf>(7$6N;-c)Ung1cpasq!Qp zwesP@b}5HVDvoT}p0OQlFUWqwuVCAvmF>JYG!mm?ndGn^YG`VObrTo9sj}#5dG}nvLCjAtO zG~lzZ=qdr5o=3Ml!P6q0&{DG!N&!4&`%$rL6>1SmuS74t3=FX3r36=j+}Z+L>j>ll zY$+t>yO~|hi>xYYG3Qvv+3%gd*}L+uEe{NCdcYqKgJ2$N(g-U3%NAGf@{F>w|}kgNLP4yj^* z0}o>UBh{rz-spor+TFyJLU}*jWu$z` zX(|6`jt4ooK=!@z1GXq`w?jKG{))!R?5Q)GWCtP}GyXKMBd<@H=P#huQtk2<%{$CW zSb06mEKQqkuHJZDXK@R@TKKX!bgOzIxV)5wDH!9)a;gU1Uh(@mACYvX1dPQ&%ZdeWanKvIm@@l2liPmY}`Yl?~v+RF?_ewwW32f%5WoTEUEr0S;Z13J_kF7R^if zrJ%BUf#uBBb?sF*o&pe)Ii*^A<#X(|3$H!qc(G>BPyXcNx|j`3Zk7;!&yzK`;hTmq zb=}7gCb+9)s=2!1q0WAZwzk7DOa5Yew4Kj%W~iEe3SK-4U@C%5&49t(eR4I(6E{Ce zI2}3}ge)$G6z>*rXxkRY0J7W-g!99d2XB}0U`5JzbBBbj!6`qfFI?cu(l4YGiWq?J z&9(^vU*_HrI9zdZ!1m|ez*JCdN=^W$zYoGPz2CpQjBUR0UP+i%iy?Y-ZS(>r*glDo zvR6L|UP_ix#s~NRj_Yz{?ZJhYN|Pq>a34*+nB=M4s)Kn?cv4P)qwXY&V`|D6@uo2Y zVi#__duoP&$G#`ZROjh2r-yC%o~NIYWECO50AgcM){*((ga8)a01X`oa?FG741`?Xq`+gk+C-gxgV@C;ex9@|ISPL0Qa#8ycpSPE?;E-;CDJBDU zj8PXQRzYlw))yhZt4^g24{lc?Km_|P3N(HG?!RkMAAa05M9SsMkTJ8+n}R}N9{l|V z=*m!G*{ZzmC_+%}hbuCoX%O(_LP((eu^;~ZKN+7a2-~M9$AP4iP!&ijf~#3k>Bp4V zqqoN4Qsdb7tGjbSBP!`}1=NnOxDP*=ee-q_vF1|P5MIjYcOVdKA6@_qe&lFF<1Ci?*a0>Gs3jqV?nR|wygdT zqH$Emd(5z7yTW^g#$Q01$+jlOQ`$+2O2+AiQ9kt**(E&lb#|OxZj4ksE$p4}t<{F5 z`4xPys*^8$Nbx8@(>Vkt6`x{pgSF z8&db=|A*x%=m)QP@Sb$D0q(f8Qwyc>d+d#Dm|pvBmY}|GgVB-qt8jhi@zLL^*S8FS z!d!Cy$^=-^LinPgMCwLM z^pp#)jn3-P>D-DLbcg1r9|6SKHr>0|3;WzYA>TSra8mTBgK10D09l?DCVGpQV#{nA zXKHUVb77A>Mt99Zs=R3QFnPkW3Yjw@b$-x~&KCKfMvUFdsFm~Gy2id_cZy0-O)St@VvsVyE_+G5+#VICmH|LSb)D6W-yO_o^gccR8%b zn@zpp?-VCnZvtysF+;i1<#@h}a)G);rxg-NY0)Hfivn79z@XzAv;q_%v#%bIR2$LC z#_Rki^XrTFMF_83As;3bt7+U=62{=YL!u}rG%d+}aX|*UC&yK!Y^D+Ry=Mc3swVm? zld4}&-)b*mXBQAP!^n7(5;^U%!3+E7V)S$wh#7S~wG0m&TxAN5z&ar83~i5|KY5mU z=m2oNyq2-><{f2&u;3)E-2b>_`SkcRm3ex;^3=qO?7C;040&JX;cNNsbFLWSXcnMC z(Jp@d;Kdt_rvWl~yDUPyTz_!_=0R0d(}*Hy{AOf0sSk~9`+X%wzj98%v&HtVL*;Fc z!ACr$YTHOuZ8{jnkl-20;n}FXdG{Ffd7^WcM;a;mMa_mpU z27#bL7`X|4f-RucpL_7u0M8(Yy9wts&jv&gwBjj(LnVLZWoI@n+PKTlf&(2oqKx*MYGU^1idQBzOAv6AMYBzu4(pXmd+ zp_d}T*xV?PvD3czGeu}W`Bx=ulx`|w>UMe#2wgL6;yX}=`y9hN007|v-rEb;`U?Eg z;HljDf=HhosC3<%*7v=Y7&o=VG5_tYAu}I1E@fX7Gmp+l43$-aNRIyV3z@bo<$g>e z=Ja7as6g8iA4G|&sf0qgP1(ivmD8?{nyL2#z89uD17BD2!wb}Ck52tZIoA90{x{K- z`M|yCR9`15LJSYXj65MXs{;`K8h&Z+^^V5bvp>c?aRK!3BTCMlFLb;Gn0Tz$6}7`BNUtCX!SX6Ks{ z#D^ZseST|34h3gV)R3N8%)3@WgXl{l$aw$-fQ4WgFImvt19Gj`*O6{t=8EDh@7|>ws7m-S-fQL}D-|m;X^p(3Tu6u!v`3f&rNykRF z{=9nS*Fku4WemqK?2vV0zd-g$L*0gMPVl))h-4k6`4TWj@lwBSgC zO#&GW)~Y8w9mCihnX)lCj-=cx&msVz)cuvR+Sa}JdZ-!{(!2ODGll&q>e+{WoxUE$OI{~GqhM#-Z0zqFpjB(EGgTA=)`sGq zpfvc2Rz}t`ZF_V}WfrRA5THELaG&5zGYo;%35qO)Q0Ed;P^VI07}&u4uE63HT%v6X z6sDcB_~p3=lZs)x2)}VFpc@ndXxOXQ4bE6wCuP5jpzeJIN-B7KZ4k0DxpE|{p>}&X zO|s}HKq8#p)1(D$1>XppM0^(US@ z{B5%cE0;!#*LyR%IG*Htb<2h9hI2-9+*o}0lerEZ1h%{OZkZ8 zMij*Czh|)V|CoBqs3@cLeHd`)1}W)o>Fy2@K|-Xv8G7t2}Qb*k{m+1 z{(C&<{N8shm!DiNm}j27@B7MUg;X$D38?A$!ENvWbR|p{Nc&7cnNgT*utrvJHme-5 z-y;nZYmF!cQ4#I$ka73&TH#6>uLJc54ZuN@KyPeoR@WbT?qHAw#Svdxd^Y=bHm2i5 zA_&Auf=|gz^I9nR6n4NsVBe6CeQ#G@I-KVSp3aw=PWTOh|2C&JyMIx2!okI7s_8-P3@TtUE0 z#zaE4kE3n{k00QS_sjQ{F%bL&qox9#x-t+lz*M`P63i2l2>uL#6L&C&REIUVop1Dh zIGUR^qe?21uI~TiP@oY{z$_sL+}fl^W+#|*YcIij>yu=;85gE|0CEj#$x-HC0mXe| zLwyiLjnWNlq6ik$zN%`2dBBkZ^Uee%g>GX2y@6EY5tu!v&TB8w=3|`SN%2qH!o-Lp z835PLh6HEB+1`UiRlW1}NJQ&wmY8F6BF|JBfHlcj*TBEN1?(P;s#2)CV0=#mzUoDO zVA?6c4ZOi4tF;6g+b|YvkPa+EASez569jN*?urGPn!t{LxcY$!Kja?p^i`TGpkAQ@ z>wJW@x|vyI-Dw=<&w!XbKo+M~cVOa>iyj4Jr{)m=tvxJ`ho!WGKq**}vExT_Vm5D9b zZk1Ijw#JPGt(=x(?8z$!pTE&oMxnS3>Eykqlh5&;r!{COAxP$ojaYI6RQe`tM_N5^_1Ne^#kv9` zyUsBc@SPap!6bM~SaDr;DkwUX!OQP|f1LPdhk{MfBI0{NDeQVPz>e^PDli&{>AkV5 z!KXt#uubj)7%CB6q0xGmaqG^!N7_Rle9lWcOR;X+2}2hhD-k^ z92wJCkk|yIb*;3wWbl-LfC5`PV9+h21vQ4&4Y5K`?syt$k0jb2o= zor)HunsvAbKni~WTZyfsV#oTo06~gv-(K4*)hfUXCk8cjBGW4Yt4~qZ%pgCcogbtq zImg-Y3%1sB;JF!>AUZ|6fG=eoQ0Eg6hdOvV@@-chXF(jpd${bP2}al8!d1lTs$d-- zsWE;hx*&Aby~~o$r01I*Qw?C!425dfA>k$u7_sm41q3rHkreo|{+9Tu(!%2yD?b0v zqd(pVX7ZEu1Y5Y$z2cg}$3%Jc+a5{Y&Uvu&GRV_qw`2o1ls$MciIbv$`db5=BvSJL z=zHmu5@kg+4@3Zr3gbmnzS8Au+N{&U4L$F?6nhKu0nC9GD;PNf%n?2dF^6<(1PFjV zXCS5cn$Qqr?I3Hq;gk&&4xFWVJSsWAZ!-wJA@Lo7T^iiV%KK9prNELbKw^>Qbu< zI^g4ub?I^g#S_dREyLAcZz94?lAxDUFe(oPUPhkdOC=@`cX3{H2!C$wV zL0*EZ8Zo4v2;N(`kE~<;j;}G`xCa?X-R-vk8#V$?+9^Nb#*K$Vn=`RNuB&IlFw8F7 zZY&`&PywODN~XA4K~#JdPKjTz>z$=+|867DSqFcB+q8|(MoZ5H-zwXP@NbGLl6}r@ z_rMWQhA|CzH|t9)vn$kG!XIE$oFc`Qi&DZD75oYFImEw(HNcfb90LO2dXHDskr)-m zE+`X@KMU={=;lPg$X|92u)TAj;lvFeMPhpY2LoSUbNffai4mzedL@4NAI#*4(WGQ~ zZ0e+s7{V&L(hCVda&R2bV8)g1$Zw8lR!1IS#`}m9NQ*pK?z}hpS;EKys46P{sKchw z^6E}JbBa2{mVheUiDxp8L*b^-IgEEN*^bu2PDy+*&Fx_fj$~ik1 z@N;8as{RR?rur7W@mnmuBHZi}?3C(l>z*jq;ZiTZF_ngND~`HTfA<2+r>zeV3t52g z-$)+T%&>p>{c#_5zF8cVjqm66JL5PvQdRGgy&jTiv){K>fMK(3Z!QMvNhzw<+V216 z#DHI!Ec4oA;142$k$O{-JVKsqHyf!rg}`PQm4po+ez5>+}k@>gc0uuQzakmv&)`|0~bDv|df8aFtjVTIRi7Zq}~ z0iEZhV_o;BXR|zl$@7wfm4M%E>Yk1JBW9mid|o3;7^$Eeri?N7QOH`iocwe1lCkh> z20FU5TNE$rzS<}_;gYTHYXEc@oNEhOnn>LLO|n}dihU=Tg()KuKNToc4P;I$sU7GF zf;zE`E&_ zHS(h+xblk$5g7z`@22xn-DnwJQ=MIb@%+#Sc#A9ckZc72Ij%_c2gUgGsk;I0Jr*P(^}OW_bVV)qfyHB+@X3MwZ9KS-O8zk>xAX)EIYZZfdaeDzPzF^5Ve zsP*QS41TEaUiWJ_p?cc7D3tJm6eoQ_-~5Mta>bv1 z;ZVbm-6ln7-sPl|aYh<`{yRdnBEhk9mmp3VVVs5&1mA1TLgqW;>G>)G9v>)pe!*f$ zfu}nj>`x?ZXXnh2XCVK1&q7N450Km2z7nNkH*8PCzkjtp_;Q6lFv3?hw}9dbpjp#& z6u}ZO`VVMC13UmI_H=mCpv6@#n=Bm(K(46(SCw_J^5iE90#(l?P`QHV^|kPPN6sDp zoPQ}}G0)QCvHL@*|MN-X*nbR2!q7Yh4V)eghLxUhd9Pyk}o%I)I-*uF` zXj3F%DZ2QVU(+voEXr zP|K6}f%7r!+MKNqKHB>aFW#Cxn#2?E89he;b}Y4lU22Y?5?8?hlkH4ky3{`=fgcWC zQ_y8>`(2-O+6{BqbE3wmd;z~T>vbMzVEY$|A*Rs31LklV@y!EL}gTIwv!C2lc*py4ls%H-n&W!=w{^gh{rkUKaXQvN`0f6f43-Wn_r zH)bZ339GMKLMR{YcnNYjTs@6{qvE;8FMc_jRbkrubh2pY{*X`PX($5h(2X)6bO}%2 z3XtEPAoXH`4o(3cp%F%m17(sHnoaz!EW48K6t1KsXBN3yiA=Ui{kjL5RE2ZJA2NY$ zzZMmQJQX-8JT=T7iDCRcc)Ryk`J)lf(Y1y~z5@bAX9}4#x$-S=E@EJXUTv zpryI~|{G86+e>oHvbFm-yi|B~ERUwWJXy#6|2Wa75Iua9{3bz0m|@FdxF zOGwrzw>9FKZFNBN04%zCaF54Gop#x|hQ^L?a?n{)x&hM-&4i1t^Q-N&^5Msp3|K_V zvQuwMw7ouLC3GO#OtJxqFCf7W<~Y@I#D*%U(c(QZX7aNTG*ef_SOi}7vaF#if2q#^ z?<>B*+z*Hk;wK%q(56*{6$3<@J)m7Kfsi!qRRguO?lYo;K5U71C%ly3TQL+k$VNB? zliapO3bPiRrJaEMuEFdMMF-$5fS4PdfCq}RC^&zzv2&bg%#w8DDNfFXtA7gO2fDy- zqac^Zwv7{@NqU1ES-qgzA{7=pwAv@{fspp&bn|W?Cw?|u#u#|GK)D^RrA_O`hAsq! zs?W$u{%M0F>0-MKoO^Az7l4ybK{Gp3>349GGF#!9uUi96;H1lNE=UKchRhPD+cP~` zW5y?Y+m)pmmoWifgbNr#fCrLdFoEj_5Dk zrc$D^hj3mOlv5(^qTO}>i;wI622mSA?_;`fY!GiO)F{%#vdV};b1qP$iuGmq@*lSWpr z3mQ1SfMF2Wq!r+U0KYX1{`(Y}4S?3{MWfVlbBm4|NID3pI6oU_R9VUI z;#eF$`CBj^0>;GCG=Uy5WdQbd1aMRQjDg$E$00Vube7tU2O%J((S66lc1x<=DY^g0 zf(4RmT+W&)=Icx#>OZP~CQTd}M!~AapC@K}7=)#$P7%;(;6cI9fq-g5hY1s$S}Z)< zsp*ssCrKY{#+IS_AY?b&S}icEKKbO*$9YaVL%1S+d`6tgWFH`r%EHg2f{C9@Ggl-x zj~74jM*@)j5(<^_w*$DrIGp^#ik~WuPQR5yQK`@{u@mE&q^8IWV@kfFP$oOk1>rCR zX8JA3?vr&#N*A69xKd)tR=`mG#t=}+Kte0~osd;~6lmM%KuM;v@tx>hqzXNVpgAgfW(Y@yk%E^&f#zX{Aj&3Og89x;!38^wdTY64G8~@uw zfpanIadA!3Yf3QNCF2L0*S9b@%L)f#=*zqorhy8qgh0_>*}4ph53(Lbb-1W*c$)YP zWDx}G(AYU&m{O$6BfO%OL_YrY#~H9VRPe)+1(q^k#n2jqOlkLoV2%ku)H1=io#%)= z)Q7>%35rfOX2uD?n`DI_7|oV1ua4{llLKfYL#5eCrUa)*Xhx{9Lq9mlzMK@rhiTsv z6(ktoU8l1wHj{D1L`+W z_{Kbhc*Iv3?(H!8tmgH}y>RE{JH9k9%5eaVeKNSz=Y@k7{L;RD0a)=>D%=zv4n`aX zSD*7MnBpDj8>yu4@ke9Bq zSi3abC|8g-sL*ZNSbvtrsJqPXO239GX^I_dTp~=2>+vu@|g6~Hd%OY9Z~c{zKDsu{j>j?xj z?;iGyz49ie!hB?L<_ zetgFgIpGam5i5-3^g1=?Q{MMr$l1ktr@Z5Y6M!Q4FNBB+z8Vi?D7~2VzSBKR`28 zVayg&(1i*BEc-QZ!L0whz6CpakL*j3{8tF>tg=`;l#Sm%KH&|tAb0ZnQ{oEWhqATO4<-1i(0-q$Gybicmm^<@NJ*U_2r|AYa-v@6B5GS<->f9}m2bTo+zwf|=lFUn%MeE!M z6Iua|2uz(Y?V}V!xe53*+l>*lts;f1f!>nMs8hcw-GKoizD+hTY97Y=gJ~@ZIIuD^ z!H&U?fbvYz!O_>U)dTw6sq?xBR(;$^2@6AgP=Iqk0P_6XN4~UFswXkGFTu7_0;=bR za#g4FXGo5uU_0zMZ9H(%?Sfz|p!Hv_bc3({S$Yk&g;)>^2yxaiu)AnV zos|VW{#Qu=j-_<4ikrv}kSkIBMd@CT3Ydvr1$0z6MKsehE(ZV6gaP$;x;fzx=~PLe zkT(aisbDCdV!M<-kp3>|Qp7vx15<})rLGn%ci$Fb=Yeq|Z_W=wVw7H(fNAa&48hh- zAje8F7pRaEL0X^I=UZTiNd+}}rQJLbyC3b~7xY(GfPzC|MT9AC^9=$lHwtKCtW(&3 z)B?I&c;Hfrj8L_^Hdyk56t9kzsiO04sx0ws$Z*S%L>-i%Ew|g3RczW^(35 zU|d;Fi1~M5fmAbVVc)A7&yf+KfQNS9>#a{!KmWzleFVIl*)?FC4G#5jl(K~zzP$LP z8imls3RF2$j%Ku|f&ZkVm13-zDi1%~6^wxbY!c{Y&M!X-*MQ)Dm7dXjzGQoQp{5*o;P8vzV28ld0ge{6YvE2%Qt0D}wn=qXroZ~W?(7no8 zvfc*f@xVUREMa%$SPC)QefmzLlWfT-V)&YnI=rmztL$KeXcLr=3gl~mG(_XB`JeO^ zsFA?b;oRSRVnBz32LZmok~-YXr;ZjoE5)(qm^aE>83^Kqjx+3vt8U2$&5Wpe_a z*a*;2ymLaG=U4eit5**=c}I*N;ua%MXZYR~^b)ZC)A=oYnMn{+J;MqDe#y3@jT#IB z$W&|?S}$Oq+uiUe28v80_l z8r)`3{Z>{UePC3-E`o1Shw~*II#0;1nworVk5q56Y#3E7csf-NSJGF$|aAf-gA;;^*(_fS6vLv?i8~S+V7(;+hQvh_nlq zRgh~4UxYqe$C*-+piL?Tx_?|@hSjI;5H(WLd9`?jS&}R~FhRUd+5y$biNNN%``%0s ztY>ya;~wP(pqhlMsnLua-W+I3>b;9hks4!h78Sd%AwV`$^biiJQkj6}_zk%{96{D6 zfC9KYz>&?#h#E=J!=urKQ=`;?dlJ@r2;$HpVJ!%A{r`pC!LNaW=Xvy@fAEo)pT-lX z6NXfI{)-2N#hy6dXoC(8s@1n0-tMy@ua0AvnrLrYdIJXcco*FCxSP*w95$Oku0NPP>SmdQB{!i3>tzQl?3pwzJpLJn7ffX9-c=U(ZKb@K5`)if*XGt|4C-e50b`^p!G5X>GoC``%WHtKt{iGn z{?~skZmx+|lAjMDMNPW`Tr7@^eLV2C@W9lQie)Uot&hqY+PO4RAXut3xy(5IZ2KMQM?2O_ZPqdxfrz`!U2=X-KL>1NZ( zeS{LFt~EZBT5jOTon$I4cW?r3@GQ^-mp|*td&ziO5W&WowW}SHOwZ6_GybpF9Tq7E z(qDg7x}lJT%36X^XL+Fo&jkTg-hufhd>Ewbe}!k1cdj_c7{*!wL3+TKJq=Pmz+J_lA^6IVDoY9c&AerA~v@VySBo3OTXx> z`@Y&6e`G9uO&4{1{BVD1bS@_1et{t5wVegRtaO$HYw35Q-frEZ%FsLPm2?n&NTG?6 zjWY`<;<=22r7-@xjY?o=))ecIe;aPImviM(o97&`>`=#F- zQ1-N$*l30M{#{@3fgUK`3Fo&x8g;&B6fMqM93+dCA3U%|3p}&CQzV>}S_nfn7h)D0 z&4wZ$+S#717@lsAT7ztAEBq_>39jqS@^sgW$;V*oO!%gQ;L z%Dvw5hSiru-h*RHO^LO4c{XL5gboG=CDmPhNzkn6nnj}i6#t=ObnJ6reSY@v|W z7i<5dw?0SEgW$t`Kk9!2yQqSU>2TPIkq5Y70}ya|W(i&fxQA5SkEV*JH2ix}u;|vl zUAI|4T{6#zmL=YA*Nysfzg4-FK!|u~Ui*0+CFW|WDx(RWi8w=G^esZQc`60F?d0{o z8MyjV#iio0zz=u+K29V)-}fIiax zO1;Y$bG;v_{$7{xS-Byx8U$$3_XE%1bA*$P!PL20Q^m>96Z4qax0 zkeNO+Cts(k2*XdklTxA|4S z`Q}rlm>TH_k(XSqzIo$iInk%;g;^efFfWoG-eCrcQyV|}Jip^juSFlb%nc$OYK-lYC&phiq$)MjyN2-xHcn{YM zOBQ_dbeZN`A`|uVK0D*1*O_j9Qu1Isqx32KL%S7Wz2CV_YV^3%P zUgIH}k9N;38=qRFW{H5W>JC1Ny4Qw;R#c>_LHqNw&I6~}`oGKLF&38zJd-uHB)cnh zMLVyAgDj7s4@urXS*}u;GiJ9d?iXh3NjskWh9>ZQDy+!n=g+QzQN?zsp{1hh#T@#d zcuVitwD;S=H`O zRJq7}8~2&WcRai8?j5yn$4H_1RA6JGh@PcLqr=B@j1PQ+)BTcqv-0p~Shxf0^rAl_Dw1eYbni0ZGlwv_oS!RV@ z*Na}a^o(I%N;i*V0n1dcRh;G3_r2g*`@KCra9+*UvNR}ejrBc#C_yb zac5P18sAw!xqUua8vnzJWBCWPspS9iit!>T!Kb_t!RkW~UrigxvBj6;BTrbdY{Ju2 zNa~>h*O=Ff0Q^rz^nuP7r`oxv_(eDOIbaf)Tt31 zFgRFn_>TB^wG#G6zNKCWy5%dtsTeka{V8g;YH_nlVEb*csZwdRXdv^$GubO38UOd~ zJ&_6Qvi<#^kCpC6{(GUevtDds&y@dm$mHNz=bgTF$l#M*=+e2i$(x_KA&=0~xeqBy zY*@rr$~H6_UPJ9dcg(st;b=RPU}v7bkx{Dy`r=rh^&To_GID^^;KZep`6 zs2@xrxVUkJ(2}p9?a_o}y2SC2Yc;Kg`j*z=39qKR1nzSp)o<8RQ{3XPoK)dtdR zbnvV6a0zw3CwH#5D^Ji-;wYo2M8DSm=5R&q61!j|z`K_4dxYSk-Na?VZ+ZgcipRKKnw*!}o_rp2Jygk2BQ2kv}x$A=ggBW;QqR zocai+clh_GISmfcnCOXx2i@8j#5RS)adv+M<;mfg+s@LzC7d7fy3Bd-Lc5WcafWkC z$=IP;=>BY^asmC@O&Z4$ZK&0%di_@ONRs1RZT*`sCxNvrn=4!ekrRfBx$oAO-dnl< z9i^v_m75bAlpcaSXy!RrI(%+r_R-uDD5R%=_mt4`nd$t*(I3d+ub<%=a<&Gbym)zG z-ZEPcB#Fwo-)>PA2f@bFVCIiT!SF21lf-G@--6iVa?js1Uq58uO1;iAm(uuYR!C*W z<@d;8**0)H*=e@3m4&wKyT0HcHdtm6Wer{RPEk%DBwHTrzuy0^#sj0@$6f2L5AVKUZ~qc=z1m4cmo1*o zHe1;{MKW}Ka1A-#9yay(`PSJwE$cOPPbH=ypIsw)ypz_E=H_(c&b`(0;F}BFLW^IH zx9AKwSQ+5!(`*Ap>qDU4{DL{0HeMtmn?StM6!TIDod&i0Mdj@AuCl0*JrSzw6~jw3 zo54w(g=;EWR2}Jc3e+pOpkqd9I&u7_lJ*XiYYP1G3wzv_a)^FkwDS)gItkYWyn?xP z9>QvBJ`BQ%OwtR=GKJQu@vwM)r~SP{*OBiexJ(#}HD@(TZ`!jkAIYBLu|#MNTXPQ5 zFzio^{AE$On4%VgjQ-E4Es20C6DSmjAI>u&jbJE~`?3S=23gsS0P7j6R1BrKAy1EI z4vO@3$DC92({GsumEL1ju6|`ZqVr#@AIP5i{!SNbV}$phfeRuM(0DHX*2F0q(_&c0 zdct{QI7gtiqjg1%@PbMIJ*CiwGfQ7GXChzHeO3<^=}F)aj$%IEe#&b@nnwyj8(`TI z#|gm7X}{WZky_I-BktcgU}23bfoGmHcHP`Bjt(UXtN4ks$7O zlL*%(3U1jEGwC1l$tM$tk>G_iZl;9q&af%n`9G42J87fW@*jwrx{+OK;GlWkwRf4z zJ~r|X;n(HP9+MCj%jB%AIPB{t&o9D94eYs{Cub}bO9O0SB){p>PVjSW+$>>{5==Bqzh>^DNp8Vsgzwof8+G+#|QNMEC(^lo1D|J z$A@GjE#>{pve;Y*QiL`3FH#S-vv2v?Ms?<~8MRCBLACXtzVB5cfAu`F*BQm%souM+ zJ5m1A(BxcLbQ=C_?nLfw`?|y-xmKAo1+vxUvMJ|*_`8g?`K5R)xb@NRgkyGw6-~9d zdz)c)wkBEzHifRYr{|6#k%)9_JP>J3NP@OZ=QKs7VnDo*-!o>T{J>vlMqeF^UmCDU zrgvJ0h-y$pP<`bhtQM*RxADB#RHfWYuO>HIr7FTlx+1Cmj?1;u?wH!Zd8PB5OaULu z2$?o70X4&0`*8bZpI#Gk*;t>W^t)4=&ofy2$NxKg<0bEK6qlTcPk0zFfSO=!5OP5# z#)#;KEW0x5zTq-xX{~obF`LKOpui#OlM=xp>bLGC^g@SDi~ zZ5p&*a$C$~d}bfATHeX5<~AyrY)SYu%)*HxgK2{r>fykWovC$slR1wQG}OK|&yuzZybgJAU)4j=T{bA_A!oEQpdBnh#vW z2~f=eySs~f4h|?*t=a~(|IF}g8b9HVg{+AQQ%p{V>jzGS8fv&p)0?B9 z$JE$;?-rRu+RC9PFSf)QBBCz_7L)vleJ{``4@5@+w|MoJ@+zISSErUMx^{k;`DVn^>;F2NzYQS-Wg zc;0o_GR=e+3MZGd=aD>T=g|CC^<*|<K{0SB&7-;XGY^v;^;UXvHeNZD+o_j_-ZsT9>+4~duOR<+L8cX15N zXCBFP{wtJ+-^Hxmp3<$6LYUxg2i>>Xl&GXoL{YF(7WXWPwg2cvbq>PSnX8a)#nBez zFpAR#V4rSQ%(c&&UNx7-__}xsa#!4Bw_HJov$~a;`Fh728dX4-dKO^t&G)VsL+@DxT?X zX-2mCs!my^G~-DU>xYucJ~w)!+pzNyO+3zT;N$jpYSxrD)h$wKcjprhH$igN?pn=W zY#&RE3X_=;UYrECjAlA7q9PnW$TatssQBLj=Lm`EV)lbO%b zQVI)^DQ=Hlilq`90%p6{iLQfQSK^jbL&Wi;#9f_ku$Iniw-=?oQ52e=Bh!aNX4)j%S>ymUT!V!V_g+-ATbq6*UjQ{n(hj=M*Pr$f{U(#8KM(YOC9PT zA+Hr}`iqCADf-bpW>l}^<*E61bK2+vnr+^SszT+2?|fFji9CPe=)dFPSPR1A#jci)I-H$ zthw!%-VIe2(D(^(Dyh%*Xf#|!eG4dska(u}-});=2&IfsQ_Hfp^E@|fD|FIMr`&jd zhfB;;(}C9IsTxMyzE_tdn3*(f!Kkts`*Ax?I2eU6kA3y)8wuz7XDdHZ@Rv(;-%9<3 z&o~1wMMobYO4_N*&CUYLZ$*^IK8N{=AtCh>c1@JY zAI@gbCQG|xOYX8R`qNZKo>Ye@X9mh8ae9`X+ZyP#@|H`bW}rNRb4-xtN2!f_$1rvj zVuC5%fqHY*O%@)D(-PKGHc2VLKP96$0qQ%&>VD@4xo#|ly?r&qod*qa{h%m*J6w_c zm-r!bTU^%9^_1lvk@v%P?1dYQNp#X)@)8C4NAF&jTJDt%99|s6owu1^Ouls8y&=!+ z)A7_cV0Ru#r!TlFK2BPi=Z0;>{jiM~_D4|=?9TYHd=s(=hH`6rV{O6^X4%zMoH-#G&q<8Ef$6uZwDU{iwI8Ulxd9$AN99St3ZD zC3zoe_kIrJuLi;3bi%B2^GJ(D&XcL4S|S3`*VLblZVwblId5c8l?8dKRp^pcP%FG+)o$x z4kwEZ47|Wj8zqe6U;dt?iZ&;WlBwkA0~Pl|p2nvy_CHJHtj^w_w8TI=be>G#{A%cI3{c6u-;#Ca{85?%&79k0(&2utda{^)iB+@*{1dIqbSk8hf1&VI$w%=T=OrS#Rw}C{HJH%--*~ z|2-gRaowm|He>3qb(ASqDnXk?Lw`ekXvwya+jPxng0g10OXSq(@$DPh_Be~@RpNkL z^RnM&MRZ23rr{rPX$ko;5Hyo8j&@2j$%ZO;`oL$q+)m2mr!fAMt&#*MbRDUlICiHAkqF9Ib*~ToPsyayw6!U^VUnR$`oQ+JYW}J+ znLen9zf|XKZW2{QS;P8$@#2qCZ^Q?LExwArdFF}WQIbiWH|H7mD!ac%76P!9b3bMo zJFJ#mPTZutcuTZ@A0J^da%LYq?qq2Q-xBKcwM@>WhfG=2>C^K>1s_97H6hUrQ-}RG zQ@eP-CnJg*RH!6FxG7)CTS`lgfh4Xxy z$pKNx6fR7Do}~b$-cspjZ<6OD6}wTMFeDFqeVDE3fR61{ui(kf1S$T9aCn2~d6DjW z+A+cy>w6g9vjL<+xTr;GXhHT4R+=R{OK*8b#$}2f2&>k3nv0dcn?5Z2#@sjxzhfsR{V(e>+0VK3ztTUvz-AjMS!$8kQqzs{sR{|Fe_q^+Gr9-?YkaWMVZEIXv4&VCgd3V+^O4hCV%Tx|L4l9%&Pv%IJD`XJFmErQCI~qhU3RU~wdCG4p zyh~o3eJhLT+s%XQb?z+logh3j48`d`pd?QBZO_cSOd$Y1-Uq4NIaJCiHu5cEDIXfr zZv9e1cj%p%KkR;j{#irdNfT;vZ+s1ZP^0%()l(rx#euGq|v=`D8;&s*`mmIPbQl7hDsK{K9Wuo@#|2hy1 zU7yizv^ET~;#?r)uqNSNXmyl6q)k*+8rsOB0+w}{twdROOqr->s%NacF9UoiYD3;TBcZ@VNcezrc6QGBlImE71SXJ1F5@>u%&%4aV# zzgEbOWvf>Ae4eJ(X79uynlrU34aL!Xg7P-pKx&-e-ba^Ght3p=dVpP>vMjzgX;LX+Dc2qIX)V=APpuNL*KaM`N2^pCyt$k!tgQ zy#5EbKBqr+nbATqyG~wv+bJuBijLkSG%K9)#B=+1S7EqvpD8h6={pBQnWz@6 zF=@$yst!apMV}x=9;1@K=z3bu-QKW26Bxu_+zBauzE>oVgud|T{J#|i8>{w9Z?D>)Rj&vPVH{8;A-TJy36N8e7>(;vZ1GMQ zbe1~GA4b(-VYlYMx&DsvyG_w?*YD8h`s8o*VVVlBR9#&pkHw7VDVu1>RTLt@S49M_ zlLx*uYF~y^rHq{?{IS=)JfcH;mlW)WDOYXkAG*U0I=o)7x6j)g=!(inG8ahq#QC*_ z9Nc*|V;o%No4+odidiWelHgg7PTp-s4TeSz3P-|CB5Q01bFqQu6; z&b?7$<4Dpa7_u9zDqP@xQdGEgnAH-BTGf|6eI8k%P3Od4;GHkQ_$pN;4&Jc6rcHTs zpts$w>8P}3F0O&83OnuIYa<28^=RZJvz8z?%TEQdjPFJ1A?&@znOaTrvc$W~3gaWr zDW-WlHM^6!6Q8(mh6?e_84IKS!=L{^moq&c>d_ z2Cr7=%QDRjuYe6mLX1-P5XzKBi=$iVlKD*d=>mbW4$;2fX3shQ{!&G2-+(;8w%PHw zD_v|yH2j+=tENf^E1udRIP7V~kH0J*YW5SO#C*h1t3rW~Azei8^pdUfk zyaayzoG3XcJ4+RoLJcLO6Vi}IB8$y2jz>nl%ba+N%!}dOs8?b-3A3}q*-;P1LlXum z`z$JdzvhW;fv_nd_T*<~Q4>3OVXmlz`=qICHm|d9_9xFpgt^hTBWmu#sROuZc<-t@ z1)h#1y;N|K+(U^R{7FJ{;Cp$-!n}Bdzx%MO2?(pNj=?xntM~1?jpdw^|JCS%bAA># z7w$kf_22`fJu#3~&e#Yqa9!@qzIHC`mi3KjSNpG)gg2czg3LfqbVd84Ox0^%)GiWh z-qMo?MiBPhd~NiZ?9Rn0T{-8;{`K9kil;3`wE4Kdg%xb{Y{gcA`p#f#cWD)8ZH?`= z8Yvh&W^!qq1&Fz@-t_+703U?cQM2z1kPC3tX84~_2ei5eV(deuF68!SJ9Cz8C`Eka zX3Bn){QG&Gkr5(eL!2kJ^V0Uqp!i1P`V2`^(%oCiSG-|WaqnUqh%e^93v2>h{Pg=s zr7$lymSKy_o59Ccd7o?3fL=y}!nBW5C+cyQla2mzE#CW8YZOs!#G;Pt=f1z#s-Xk> zO$`cezMIH#Z-yg=YIB!6k!FN*Qb zwxRBQ?~uU&REd~j)horN2&|1&RDmg%(D#mPkE0Z2XX_W%S}(%VzB@MAMdIts$7PC7 zxB4>Fr`n$`#1~3Y=S`Y-3la=IxAgL394jWOStfcVN<@<-AZk8Y)CiTc_VY%`_0n1q zH{uJe8BG>gki$9>^Q=kDUE__BAMv#_b*gEeg8oq!3TMNsiQ;J80@9a_VIBnl`=g=)6&a%`Xw56ZIIDrR_K_e`Y0`z8NiIEyPF% zUx$mK{loddU@1pfF;DFjlV0BLH=&rG??tAHzxu0sskxxvWRYwXK==P29GVPC#MB_P zz@(sq$ENzvoVqy{deyFZo$^KzQ0@>`fm&T2{_q;d0rc&UFE!%qljzNK(*f8*97Ep1 zx@;ro9E~(F0^9FpsIu|KGun?U(C+n@j+AfQM$AmQ{|GTi@#<9WMaW5Uer|DfB3=SO~N4G|g`MqB9D_61K^+U82zx$1}q5cjQ@Z@ut zcheVtcn}~L`n~oo%Jp~?aTgHWnRkn6KhDx)pkv;2=o>L$9x+URW+f#0^8Xln z3$Up6_Wc(SWautI8c{l=bm%TYQ9(Kcq#K6r0cns@Qjk!P6ai^ax*Mds8M@(oNB8@l z_xC^N{N8h2Y%VsNJ+s!DwZ8E@_kDk?|4;zASo^Ea%O53i`!w8Cn>JXB+bac2)U(`d ziQIcXnDaEtRG;&F^uDG_h?khlyhW8c?#eV*B#T7FwtyF0Hu=KSpW)H&E~}_cJE-p~ zC6viBXK(bS;a0=uJ8E}N-8lp3UeflAq;r0mEpRYE3s=?|_R-(H`DPq`&0(dU6HlWN z>k?CWSAUoXQcoBxt|8(bqNdM$^5WCS31z{qdjh*;8z&e$43Y|BBt49ociGSgdQ+H` zKb<&8l_rp3My;-PH1op>NkJpXRmnbuoNNi!dMj)ky1Z*J_3Kn zJV%NuAJAg1aRa+E*jva7a}f?{ae%XEIhUdSL#*j(%de{ z!{QfDe(V-dZ5y)%Ssl7a(;7~dxQ@>0U3INpKl=f(%hci@nrYy|)u;vfBq|~EoZ8v1 zpN~F6&FF-Vb?`6cf`5ry@tQ4Jr!d1M6K zhTuUaM1|G4TD+9L*R8`Ow2Qt`&M3nhq%t0dSM!Z9thN$yF6y)gU+PF#^gVvb_CjHX z>g1h;5zDhuLx&t~942Yc%0Zm30n&7x88=%bFq)co%aHc_WJ=_5H=MIszSL9IHxhFl z5hWv{+9MC~Tm~;I`0_PzK0p5v<3Tzaz};-p;&jL9vEzAJ@4V-Ar|Ps>T<6(NA%l0K zk$B_=`vhN}kSlU=I46aMr2itbON@}p#Ux~q8_2=J_`PQe41I|}gSWQR09eFlbAbd| z&=-b=nGY?86~;b;3{qe;VCg)Eg4LLzB3L-E5bPkg%+mB+j66_xubw%7x2?Q248ubE zyBEjo_3)4bApKWBN7Fe;Qy1wuo)mRD_^=GRU5u{L?#2u&=N$wMzbw{3)0l17#bUu( zctJQVBGLvzA97UGlE>-z^S!3jMo_0Ib_&JWlB>!-O5U?>pF1P+on(Ex?+UY04mHa) zWi(k@+ZuqdAG@cbcP`V}KswjHk^@Ug5mu$zq!_AJm#_JDx zqLS5bjpR?SjBoC%kI&SCoo>^~_)B~f2EEN>?RR(jHu}d2Co!*h3~>5*w*!wrsa;ok z4mjzNh%JzZ^%mq~OLZo?6l1U#y=U=c0A<@on#Ov!9uX7bP<~WtbU9jXb53RWH!&)y z@^zfvWz_5W>|QqC4A0;|$K%N|T2RL}C%PZQF5&t5RrI96*Eza<59LkcIs)!ugckg! z^0yHhY4fd*VoRS+(m0-da}k>Z!h&-Ug|Yki7f^#DvrtD%mK%jnB8Ukmqq^^xuChOo#Dw-WVICJU}>&PB# zp-&_O;2Xx5fFw9+T|82cUX+vM&fs9b(3IHrXo0QI>z z|HA<#xL7z?-`)&Jt@Ng`gPhJ-U^vkQ9zhfBLw~{^fMSe;BqTi+D+IB2NXOO{3~+cC z7D=}NC$upFMbxjy&>s1tI~p9aWi7mhR?Vat{K)tAZvNZ;FwntzDiXvf%`lPjV%F6q zM^2>g5K<~chpahO3879IzDGD{u1|bP#C=~SQGDDMYG9`$-(WC7eMP~a*lw19xm{k* z?JkqPkJa>GwVOp@r7mcilf_=#9=rUATp{y1$9;`k)GkL!R1eA zWee;_@gTrRNlPmkSVyD*mztrt?)XDcP!JNP?lJ=GPs#wyg`{o;oC{bE$ADDPPT(KN zj!t5Gq10b#Bp*c+3t~Dpra{WRvOgNe68Jxm&Lq8Kz&=2Fgd;Zs?EBhJY5=kunUVPL zO}|7Kse_o*1n@O2_kV2vl?tK8j}3bThjgHF6Kii1m|b;Ss>Qb>pWhl z_+93)>9anwZ=v5>m2P<9|BmnJvgJeDQ&tUH%Ux&NLTPX9lj+^%^Lwb20k@@mkFE8T z%gn+Wpbn!Zb7IJc;wLL=ZN?4XkBo@adacoGPX*hV$ZpI_%f&B~7eZFK&K{-Rpv$AI z1*=Ncz^R+P!$0ResXXXNmEwA>J^Id9l*TsLEHiL6H6U&*@1nkYr`_>Lk7`zS?yi?!aEy=Qa=@(H zR;}SvB6PFRHqDU$hEFL?yrB)q{oV;ff0ctt+ASE!A@ooOS=}ROJTmnlaCof@ncc8T z1ehsVGa*Slw5@x<4yFTXB#aV2Ut`jvxROGd0cnG>_=eIbVlEr;NFP4)To%zea7rZG z?sjJ&y~Kbyl2ZU82%Z7*9|Iu=URV5scbC@}yU!|Z_4LpgMkp^p95Fk{H;n_Spc^KH zg$yXt5^zHN`N^rvt1D9wHoXs0m33rb!akQ(xAj(Bw09Uk7T-LCK##Dav0 z!7lwi7A2#-^L!?e4yWcidvg+k0YH7{Mz!p&N^ham#OA)FS5;O`hFXNn&MJgi$(wDrb!&66-%OalT=<|WkK?X8^w`q@-U!4@ z8#FF*(+e0*O13OsgN$}R7j74dwnXZ9&B97QT#>CFPo5f2Xga-=d2CZ_-h6;%#6G(_ zDPzVKy|H~meXlJkV&u$LzF|cFhoELUVGg#4ms;d8Gd>%7h}d&|#1OhjSm`#l)yXx% zpyOQSy*&zVctHwmsE}Nna`g-7IvWx-tzPNJ={0y2QSd&y6j(z5cUPkhgkJZc%FSRz zs)#qw!AY~T17~d5bIIiMo-Xfuo?Y&CB)4&X38<1 z3on1iW24fmT0R)jaXUi0x7oxHFwyfkc0s6FkLg83wIs9O>BSWP+43?x!CC7KhtCJ` zW4xZMv=1kHPMS-J#Qm!`6?)-yp}9lF-rn0pDtlw8_bL?|y&>Ohevdw+gk@_}*BavO zjh$=h9`B{2?V7ZBQlyuG@W>(kd|>_)aIxW3|A0R};aBnju%EzMH~RQemMnAvc+QAm z*V@!FTGE#SQU0i{7)>Mr+%h8ol%v#}DuS`ViM|06;F|J_Ff?JD)C=4{pAGjVuqTmg z-kdbtSd@;uL768*W0!G;O#xDrCiZ<^3PE7ob6<=UIuyLFJ(=+|4s9^-+6yLkm<8|# zZ<+<3pYD{DE&wwT%>w*pX!BQqDn<^&YMHB!tiE#$ARv|>skVV1D#sm3c%g&ezXlHt z^$QG!nkKXdc=do&|Ek2$KT{Mpy%hx235_hnF9=I6(cLl}9GJmiN-NVy=%0;^xm}cQ zjO9=DF$TdzEHc<_^Je$uv+lPskt;S;sz?t zL_PieCWzzg*kFv6$m|C}ZgahXd8$SmS)Y4b_9Re*aPn|> zxbV)s9w5{|>y$o+rv7rOk)eDOWcg5ND&l%4@X4XRP>8WP_024-fnNw#)f`^=lf+Jc zYTPGRg6r<&V6DECO~c6%ciFCDRI7Wz^~g-6gQ$?Byxoj5p7)-FkT)$JV2HpYA&>e<+bt}+@Q{NjvmwNn@-DilRbW5F zfqW(lGMlPWr`aqmSadsj_!cJqnk`{2C8H_uG9HP^41n;9wo_8dRWJcF+f07@!3lK% zIkaDcbwZ0^$-jZ)twP~^!aQTWWGI>X=WGZ8)9fs-TxeSfpucUZj}yi*pfRNiN-;=l z$%^xYmJ=!(LB?>6PQa>y&l5~Tpi13B0Z;7$@vTF@35|*L&ymh^z{Gy)NhK{Ir)8~$ z3_zG^LYUctG~N2!h;9t-pmaE^Krbdch?WG=jI!J+mZOF5#R#Olq-)_l~RqbIkb zpR6o8?dK9##C3%^!twNz`4aXs&$hG&(zA$%(!Jk6^C-!Od(0{KRlE!R%ZQt*@S;Fj zq2C0lacLn>)>m&SQS$Gn$WADdV&>)BUy~^vMP%J3_$A~Io#5{xpmR+GH?KlnC(Gi6 zGwn!&D8XFu%sRczW=-Wu>|+W2TBD?Ig;%xYUXfmPJ4W-2QaA8%d@1bzPqVQ{jAg0>FcbH3$r=Cwhm-G60s z)W9b{-Bo3O^u4}#Kr5$8N_aIy>CGwPx4VbKU~5`P0uSwbb`t$Qt#4z`WxF!L)&WM2 z>O`O;Ei90M3KhROT_kw$JU_PI@=xt!6VO)_wjy zsL7pM;{jMbqFCxEu6>)$PhYwa~UX6;g2MO?Yf9!nzFjbt6fg~U-!nej#pe*Z@+??t?;GkjHmmV zR^DR8CUXDys%Q1^)<|r$P2WKIWVlHK8B>F*plM z(up{thcE*==%mTZUFaDAR?8mfy=&vgo&|KiaFdnCfCMs7gCK;jg*ySPm}>X}oJSwz z@@{+M4hVud`_U4AZ4h#^0?UG_pg+N&NZq)J#aEdfD$$2P)3BJ#BUV?9$$8 zH@nM>`hM_Nra(a)xcU;!Boc>!bSe0Bp%DnkpBI%2Xv+j^>Kb6q{0MoX7oU5p^)m#M zyS*_glDQ81qovI;G<@EH2h{p?$+0KTQ^)0Jj@V4Y)Ud(;ksK?(7}6G-(M!$%~)xVL?Z?W?-kE+tk~#wwHk8D zpLsHeQPF?*tdEGZIei(o{r3YNB?L_*~*1y%xs(-y@yvc#OsEY!^7 z0Y?ak6&si!OK0Kr?pK3? zlg~yApUS&El)|A-YxEqZ>WZ0PU)aYa;GnQ*!4b?|;O?a&CN_4mWJ)cs?zehgmQLMF zO>I`peZbc;G`=<_96c~IQ$BFJEv9@7dimY`3hY}q^&q{3*7GAceR?oLWSVxlWWE)o zpU{k9OX|MuaDFU^xo>?=(MEWE1~xAvMqhebOcv1rRxE}_goA>VdeKjw4|nEgdpK_2 z8xvdv1kTD2P22=vLqKXL4c-3FDNA3vQ5&5rnRSC3qR+o~|6Tx%6kb8%OZUagoOYa=3*S>}XB`+hKCEnss#`_`GvlV}G3i9V0|(?4!off}|%s{r+`9 zbHqz7ll8O#t5fG!^%ljdB#>s%3om~V5~98NV>v-MRqc_$*W8xG$>+v@>eQz^s9w>( zF3dq==eZ1=MUa{B0VWYeCmAF3AjM;<)3z4fN!R=o- z+}e<@glc&H6gKpkci@BrJP{Y5;Nl2!rnd1SN;=^JHj~c|fl^4vc)hf&!N!>8&?05@_2(KkNR;M+jo zun(Z9E~3Libhld1!Kyp^iy%zA~s5TgM+T|au=JsPdaKNej^JWa}Utx z&K~c*vmGpZvb3SL=}`Q9KDKg8)RwIt_zqSvtom1#HPRnK3Qf$5c}7ZSB<`;P&xAGw z>$n)BfMIMevccc%VAoN~VzS)#cDHM_bCJTmXKsE^;w>K{AT!plhDf6QSMiLytS4>t z?{VSuWTHmZDxtP+fjYNGmZE!=0)in*WW6RD@Bq+l?K&%{+@fvq^rRyjB@to&C(GnJ zIz9Z*ZdQUjzR67r*7o6duAaP0)iv{^!RTe^s&$^i4y#7UVgKw=fw-m$0ZjnprvXwc z?MCF63@O%t($h=%2~c<$cpts+Z(oggznT$@@;0@C@E(yF3_u@yof(jEE=K83-G7D> zm4QOpG`2CE*OFX1z))nLs-ki763B>$jAzX|ElD2gX6Rp&U(6VW!>U+DpHKmj>^{Vt^m@rc+zm!}{ z3>^$4+<5r33tR#=!y4~!{!{5b22f)gEYRehJ^?^$XD-ukfo%>T!KdUio=Ijk2wg|1 z89SPLqtI9Yp=3mWa%QcierZ}nU5RJucKG(6Rc;yt`kWy(MD4m|Cnm;|xVq2Cpy$3lWD36AvlqsF9V0Vlz`ao^P5e>p% z*%v-`E>loR+aGS$eURs7wHI)S=(0>%!sY<|t?kz04fZSGixn$^YFrZG zbN=+itjwp4;qL9TspS+t^9gZ26D_q|563vB0ffgWp}wi0mhK&s_>b%S(hLHr(V7r;_GpDyhsyioa9762&wZ2pjgU4dP9 zA9GRt@uby009{$xy$-E5bO!Co?Vj*@qFkymG#dum-KlDgRSdZ*Gsq>PTNY-9%<^?n z@P$+oPQ?aN&LBh7Mj1QlErju0b?yCV`3t48JB0xF?JXB&5~9jMU` zQCw4$oBYAOPwWSoy2yDnBO=lp_DAO=L6ThmCOfsxtAz`1NfM_$`AJm4<&Sc5yE2`& zTEdT+c^q1H`b#f=BZB0(jQ;E*eq#DJ#OFUYd-)nBV1AxaD7BNKlY40KNjm$>imV6q ze3LYtSlI*fKcjby-vRm@Rs==>hLd>$?N`h7hm5;!R%|+E1VlmVNPAI?UmlT!)}nca z_e(I&<%;U~Uhc_rYPY_oMFygiqelw*`1lMq`1+oKsG!KKHyK0$cZ^~f0-EX3TWUz% zts}i^d9fZ7d?c&3$J-Nn@3AYVuM2x19qQ6LcQ>4-k}zT(6{k-pG&?j(9R%VdTfv~r z+>^x)6ZHkyhDQWkP!GJlN&vX_;DT53+^M@h-$aJz`DfxtYnef+AmXARc9lxNGM=lA z4t5|F1GmSWr;y@bS=im>zWd0z|H@D|nRWj_YUvl00sRIXqxNO+>h4@qUNhi+;a5M&8x$w^qPLwsk{n=&x#h?-n~xSkGF4) z5i{(Gk9PLaH{U4#+x)nN$`NnDXPAn9J6KrsZf(O3(0|GHb~sr|J{c*>c%T;2Hu?Cg z$veyaAKEc*F*yFM3i8iG=S#FDtz3vEceh+I>eAt9Fw*K5paf5EIbWwh7 z4v>f~q77m7t*wN{XCSLo`+1PLy$bHk7xblls(&!1W8%lWpndt_A2rqK^cAhRg-#SYIy|@Gcu$2Akhu9&j<5ddb^rxMJthi z555{HAnGW`MFlFT<26O*D&{6?C{Y%4g=%Osa()b%b=zT%V#xXR?^lchI$J#3efHJ{ z_F3QYSCgZ~`M+l6EpxU?I&e&&S)g7}<8ptEQY{qk&r74C!X6!77`qn!G;kIH!fq^V z{E9X2B|lfbOzB%;4dlBf4xN>K3T2iD!HVOjtn4~WdEM_f^Cb$36_cwtmOlk?vJQMQ64#F&;5uJst`!A}7{b zpWxyx{0tkS*bJLfU5#kw=zycuS7jk?N9NRG2f8)}Urv5h+|Wk2R03UR2WZW{OaE&1 z>j#10AuZZ`$53ZB>3c@s^0Rp%9VPIFQ(HpRV%-sK?^Ji+izAm9vdGlRZWTJXUpnB1kYf5)*{I3rb(iZ^>dC?GU?~djG`hD&12O55ffuy zEakc5I?Vl%RI*Yri}n^CD&b>aE7W?_Ks%&(*#D#y>I6y$SeXF9Bl2p?xkv#uaK%HdEtSegt!M>kIETPSBeK?kD#r$W@J$w;O;EvKR9t3$vNHBOG+$f@RwmBjAcxl>U*WU zvr?En=yS%pp#^jN_jq4LGIzJ6ivKkM^PB)EI6_0+_5~B$RaW z%GnEsW(-p{8_#}8MHnpTkajP-(P}K=`^mWbMA>Q;d(CSDTLtv zb7$JcV^O1{n=V)T_i>^#f_AFE#KA-6xgB)+>zNl>-6jpscf_7YMgQ4lG*WAYH!4@YrB=LzBG#0Uj6qK=_2o-&clU|fr) z5y$+~K|qeO^irtF#Yn@~^4lb(os}=F(J$`v{y85Y&J87{TGB6iw67J%{Bs=^h0ct) z57mXMYLjxC%iLenph(5RxSLuUgx`0E`<782^5qOeB-9gEnbF-_vrCY1BNjM9GEbRo zLq=`IndSR*qjXrUw-eE63rZGGhV8yX63RmU7?!Xk=SmHf5*Q2X>9?SN?wmgZ`T=sT zsI_V(dgs@LHP!btExmy&uM;0kC0R^}`!58WdK^Ev5hYi^-lYor3h3V>qryICeal7q zFbjOBGq69iwZa=T{|PpByU*WT)ja2ZVw}yT7;Muzna=;5O~x5dfKKNWHx`nyOo?aL z;D#)M^)-SBji~dMFpmxy-FwbF#eZgu8QI3Zn%tHNWOF!P;>{!)apEIQm6Xyy?c%B+ zl;(rnTDNC$!P8EPzFW)=SWsL~W_H2d1mLObxWZ&0rliUGsr`DvB@I}Hlsx;6Y<0$r z^8DEj%kQf__y1$}9Iw>sA!5_|E5<8*$= zW^=LWN6))OIl4F5lPPq+f9wWL0khOAzHy2yo7=5um8^s-bo3wB!+%@}1AquU>Mk>a zM^Dlnc2e~TQooQ!vQ&t?xX8I!8B}GD2XVSAV*U8h?Uk=36CHb>o8Cpi*Aa!Vle8(htp^ zBeN6>3x?y*U%`A3L{C^a6CP(!O6EthFa-^9BZ9>< z28NBt57pfJg}f19H2(4rZVfNwqEu9^!`mX$ZvD}tfy-wA6cpo{#n4q%v@nT$V!c1< z95HL?@f3)bZ)eOS)7fvuNbws*u^LT!nVB(#$>W&f&Xu*;W|-aj3fe)D+>QvAPp&QY zGQ&=hR{nJKr-)7xNu!j=pF5@M7V4CA7RnaBW`N0`{UCV5RS2_7(H$#f7l8@2i20 zw9B3q#+WVmpWDL^oj4TCcIaq5LouKp0Z~g-V!V7-^SeZt7Q7=J3!+^-Ji0e^$d$!k z{u;GtE6N+WVUb9M1nre`KDiv%S}J8G@{wO)_XoF{9waEQJwHo_*(qt>p5~wRodntd zp&?}L4ay!57K|O$%OUU2mZyd=R3TlCF~0KoU?`6p6>;0VKY9!+k`^X`wj<>?TRFX~ z1ltj2yc`T=Ubdm`s|%nJ0mentgh_g+yT_gAH<`-`XL|q>#UJkl% zjf>cW13zJIG$7t_*Ar$V4s}D0i>((Tus@d9P%M8)!?(D2Arj+&2$!!A8Rk_2Gj`@F zr0`J&ZT_-8E%dN%ad`ex_aF+V<5i3w1yQ3pHHIPgk4$~RgijC3!+m?@^aa_U{ISRw ztAPcrNnaGFVq-Rz%xTX4+MW{voWYE|->&kEn<|NioEIL;Lx*%Q3(&KnMW1;;W`jak z(7`dpKN&VA1BZ%q;DUFxFLb|${5Rxd$qrWS4lM1*$h(^{?%Z>&UBnvv2RkrTMu3;~ zupu$lj~-E^WmSzuYJdMc<`2*=b)B$D*kvW>cpIo?B|}2MuZaU2O=gspaJQhW$XNwB zu7!b5Tg%9Vi5TI97X4ni^MISt92~D4Jr=(P>^F;Q|JmHpJ7&dz^f*=!P)F~d4}m_3 zgMMj&%ApMq&sp8qmMH{NKujY#E$gzm4jR+y`ac5={r@fVHzxsZiixr4Y*th|W`wW> zpIs{XYPA50(KNJsacn{ErSMOwz0IKPC^wSA)9Y-07%yyGx(v-?$?A1l$^MI5Uh z=Lt2tU0ltqz1itv6f)1*%PgR*F zF#-jx``sEU#an4+R63@{6BfFfIj(dr197g~ojc&gbfx@uRk@nlpZs&N{*KLmz8lMc z!7akRln)C=fV#qTkZ#rS{_kh?_j?s9s?REDt1a+m8#|LmS3jq@k9CTV%4NL|IDqew zT@NrGJOX!2x(|kk&p)>if*h^3W1Uko?%?k&#REZ<_s}=}%~C?OqnEfEqb_1x{kub& zV?w4dv{!o zINSnh6%E^&uyCT)FR-y`>unWf;_fLjKDdlifDd=Ds48_ns=tn#yHsqXLi5VAhZAr9 zpI71Uz4ag8f8zK0*)&zG=Nf_j(7OuG!k1RDu7G5BFau#Qew~5!$w|08k$!5WH~vdOj;~+nq9l zw-$4s5V80#Z>`{kMnvjs;zbn04|pHoJf9LF!Z+xnk<-Bc{w3i>lo?NjQ^xD(3Oo$e zu)Qn|*3SUFdU|%2ZcR!`$_zT!gAt(Ce-HEwo!U2n4jV~GsA+RrYG(k{Q3_vQ-CWL1JI$Zz{p}-bQh?99ww)I?uc}xi`M8wa@XHp^t|cVla0Ij`%wLF z2j@pkbb7v7Yzk^xPa@Xdfa3=pV~9yY=rt^M0~m!NnAaNRj||VZ+^M2AP7d)8hb? ze@JMLXfD8oDS%U+vh>;$poIHHahGi+{WFXIk4NS|Mi=n=+T=!1R$Kz{<)hJ{4dS_{ zl9r+kgH<6_wa5_cmJkBBA|k6;2`FJErae@`H0gi47ylVDo}dF}0RzA@I0=6O0rQW+ z_U=1+qZLSFK1Y#~rh~%NuYtIs+Ic-rMn>ih>M6jA#)EYHaX~Z103u32Me+ii({nFC zC{QdpxOe>aJ__qG+1%NG9~9`*vEsS(v;mx&bomwG;Mr2gyB|-0y7v>Hws>3sbzvE( zxw#*%#m>mXOqQBKt-AQLI1B~ZM~E>$dT*Z2n;=ao8#N8iS{cMpf6}h76m)lWplN~A@Ngf{jC(?u+eI!x z_p<|}-XA8@6PiFl$&YUYK-0e$>ep%SF8%;FvvS`p8XmP_-{T2$)uSFEF5N16&;}jr zj@c!@3aobbF<*kWy7)_e!pFspH|yB1+@|=r-_M)83_k~IFFSO9bDJ@w{GM1XZtr|Ig|ih zAbEBQpcc*Xf-*EGN(da@nJF2#lwwyuKwG+!07_Gw;Bgju8wZGX18B?}5?eu;o;TOt z{z;tZhM`SBtMGNl7^nmv^>H|oQ^OB9w7)3;&`ELloc~2DwcVNnwM&F~=r=;qSb|aA z`wyeqmPP34C8Em#AnXS<(w!=Zl(t~Hr=t!VhsP&WrT*S_w zl>kwqtu82|l0Du3QQCC8#3&AIaWWwEVTg?1mT_9X$>l8WYg zUK6+%z&SZ?PYP&|Y~6t2M7vuMyL1*%JFg~rIvWRLAzsU!Wa|^Z9fC5rsbGk(Yi=20giqT9P)=rhf;)j?N)Lm?$Us&l%>Khh-W*8w}21@_|iF(e)yX2>IXwV=dNM8Od=v#8;8_ysZ z0q$~a84x-6KDa zj;^mMZ>^wkTfy)vEr#R_qH(GC-l)B<-C>Ap;bE=4$Lj`*>H8p{>~no){tkiHbSIo< zs%H&#%T_Z8Z14W-r$a$<3S@=o1fFo7P6TnVbtwTkxA?F1<$w z(jL_MS?v`mye4uYZab-@99rt(-Go^4E?c|1#eH@`Ja7f=@7AxT$&E8&F{x&%D4o2t zW@_D!#`}eupG%{JRI&wG!d@}y3MRKSQhf{x3=FI=w6!0?9HEP#a-o0Sf}}oN1hzi{ zX!Y%1VI_mR^%q5AQ6A70wrg597X}eb7n~?`i_Gt{RAQ2|~nn5@kh~C3HS7^JFhkzKO znBsmW4dV$CtMYrn1%ANz{9&z^k>Oj!JOE}7UnH}x;3OWb^pFHae>W%mI-iCYXm4AK zPv}9t*y7Thx~J`L813E+0HSOFAuymnMi{d<*$QjynvOs#snH{f`gfkf|D2ltbJo^X zL77RCyKlT%#E^UE_B@_`6lh)#NTm_osWNErc{KGE^v2AWs)Ca-12uV}ws?UdJVBQeujCL}Z5z5~v%TV}M@0GKSup;0LXeuSWg}Bn4GsUoDv6Doq&-W8o%d@x@#7Cj_Cbf zsn9gwPEwYNW-m3kQvILT_5b~6dLl@pC$WA~FC|oKUeO}d4x(Yb6ycIv=YBd391p*Z zNMIM*)cZGbrXo%&--IJ*e!)?2?Nlh!W827#;9!`rc~>)%6$l#_aotM(tQo`)SFj4I zn5nQziiB7_hhwq@IXLlC*he~sZV>tzA?gKIqP>|Y_`iScp|$gb;-ON~%wO4r7AND# z=sEF2-}iJyY!bxQ0Y=732*-DBgL-S=vj}aq>fzDVX^F7%TxuuER^aC1+0MsqZ0d`cnWb(ad(6i(SJBBkHLbO#3u;b&CXju&w`{1mG<>O8v7v8#YI!`yflEa=4K z=ZeX~ms^k8Ux>6Wg(uwSTxNg4@0#tcf8A#1xVv*V++&&TdF(CU{!@b zc=WzO^$${Z{Z1Tni^n-JF@~?o<|rmAybE-N-L$z3YU!!?&0}lNaFpuNf?&& zX0fI&3;MGvrB^~TSIda!^ev7od-5C;_%s-747SN%B*;Ae?sDKE;E< zAVZC#_BVHWF1Zmdz&>(vaS+9x&vbqL0AqK#dVl3zR-X*~AiT^!nywahU3htYz#-y( z8G3%@EO!I%ea~9F)T60><7zupU;aroX{POg!};H3EH4WR1F^dYUf?NEAMt?3>tb@9 zIg=V^6oW4tx~b=FJ9R65DHWY0>WfjdPwo(-YGtKT46Ca6xEq~)_p)&Ve^z`T1$OGmtJtyaNy~~$E~LbkH*ujI(WYwG-j? ztpmu@n_bh;UBa_HU;n8=gu+ajsq#Z})$taP)RrjJCx<2`YdAW*YdX7=M()%aeG*(y zsU0seS?zNa<^@H#yhE`Z_H^qHg%SDnv47;7X@FVprf~d8XdmeohJV^PRVBLUgtsdyLz%(SKHYi77o`|n%rHbEtt*|-5s+X z;i@`aekZq;{iS3%HmDwJav8*DvQ{fT%X`8p;yB?`&AYMx2QlOo+}Pw@^Ln`d#6(nY z3hJtfN{JJa$eEhA`tp2YpVZI9J7FRQd6O(8HFB5d{d&XutO$DcX`5Xy$GpLfW=1$- zmZ-*VR|t@sxy3KEUUD0}#xAvpINnH5E;C-bZ$+zb=(6U3R_V|P*4Ml^N{C0nslAjBN@nv^*lJH( zVwgrDM7RV`VT1Y&K6+re!dzu;F%CQ{F6k3MR-+YKEgV(SGWy+RbI5ZG(HN{?rKNP_ zZnwQ2soqs3Ht=>+j&7x|iWZ(E(9CTB2uR(*uRWVti^O~1?ottcA}X6K0sF;HAfGub zip`r4Ug_qS;-NNM$`CxBzZ+pc0V#;QtOH0TKKxfu6E6UU8dRA>ji?=jOV$cWsVp7^ z(lo;5ou_}exSW~~PXpatIz8$NN0;qg^8aHqT_xnNOwo(w!+N;BA2(I)-BpK~ z;+Kt6>#KI|?5%jM`=e3rvV!(@mSrTJ(6ifhLv1^DTbY>txwB7B^{RZ1us6A?{>WsR z=UC7IE-Wn#ZARHJf;CTT{G&9e0fn(V*Sl+#0i+^$=?sq{Zwx$AVph?7QbHmGNhoc z9`hEgAn)b?x7?%D(k3FbU4R-=S=|H(!x=cmHdypmG)D1$1OTxGb=jVL zQP~9RF}T&915l&7Dc2F_!aQIdG~~{k1h%T{rSvBy4Bfw-31v|o_g@x}Q<;*^i~>GZ z|7-~Tq|fPm$hknU)ljD7t&lp(F*Guyz3-Qm|xV7L7Ak6#F86%kq-IwcXdVnF(lfgd+$0xyJF~i3% zw=YIffBX2ybMKE)ue;-AZD*?Pg&gn38knh(1qlaE*ZQc=tR01QzbauH5KXE>#Mvu> zaado0Uhh1DH|(CjdfcglxmLOtMue)^KXG4v%8-Y)123o5qYl+VLtz|ATG2&I~MU3Ywj$X+=zThF~+ z@dE0(SR$w(XqQBVk_|4JV! zsf4A#&Pe}^m~r|p@gVVFWW4id=c`pAo(tFA-<}4I;hwX*US2ypTU%S*Zb@l?sGe$~ z*Y>A@M(>|2i_*zy;=VcRSfvWDU^B*$o;#-K)Tw%M4NS(Lz`oFN;U|_it*sq{u=(rO z2&#YvpqE^`1Ww4&(UOwmAhP>~OhlLzVXq;bUi%%aJRPQ_#B*Z5=d z=WTe+Jg7&!-x?*K%H-Jro5=e(FYn{?#x25^vekYoe$3)l_@0-ZnAO+7>_Y9s zF+@>JqGC%qQSOwY{ZVX;#_?~tbuO^6t2i<7GtOAhM)3+} zzx1bNx6?1>wLOg-%Lkm`uneHpAaPg=pJ``1`1DwS=JQzj@ETBcIf45zt=Bxh!M+2xz*4K=j=!wHlpYga z*bA*DsWTkq*Y{5V;m)FBT9;&!ll*og9$$RB`b^f2`nA*mz(m>?P}UG1f{j@sK&UAJ z7u%tGwrkX3`P7duiBVQ7}`+e8RtC_~!wFgXE44m8axjy#1 zcvkLJaCl~_Y&bbqe64*2o-EG{H^)6c##(U42K>SwPM9M&pO};dsSnRTG`KTKA_~562X$4%u!DI7BF^y zIEDiI(a*@>GJD|QpVTHDEln@-HSbH`e)VmUazY{ZPfD^tbM6K+4Va*e1X^{kZ~YMq642@S4$xKuKmT< zJ@trYDY8xjC^uB-e)#qt+~F?i`Ki-8P)^0UhGOgme(Vn+zGIrLdr&x?EI~C@>+W=g z*|nD~AFTw(rPBRcAGtrWoo;K&L{|kKv0{Uyli@Roa%R*GwDi? z-U!77WmKraFJ8W9^$Rt8KEu%~{j21G6S49ZN$Z7ED#qlc_p+F>KQHYyT7_>sPvlDdx+9?R3*J&9K z%$(`dz4z|!wbx#Ky$XLf>6{{Hk9fycYSdUTO7+qM?u=1WnLF<)dNP^yWTILcGRo_m zT&3Eq_+%$_y8am6%HuVmKHcaZxI0~~&`m6o75nog{rmK$W1IU9)dCyjQVy8_82s(x zF|um8l|uiix#r6{`^eih6Q@x4>-G{dbimeHSH z6-l~6m$J2nvvyoMlj9n2d<11rt^4m2^qtnAn&~5VMNNh!zCvF)zU(-kzsBq!8O_LV zd6N>SH%RGpxhNu&Ao6|nGeZVPp^g7Qw1AB24ZX=s?FB&{u5*z=Yrf&IHRtB`IBSLW z*%2T`3P;i^z0Yk8MINukgTFbwc-~)58~CTz*@weyqh>goXME+lv0%{ciht7n7WqtX zHQ|?dE0oORG`~PkKk0mDv_Z7}Ld3A?a)3YzalSW!<)%hV96!2l9Qn*U0qa)8>1L#p ze6T9`Vnx8D`s43-+f&>4p$z&MjfV;&VYAyBa$%^abnOQ85zTXB0_jw4M3ul$WQIes zi<|;HZo&yQ;*0G-JDOqOqsECaFa{w9!{vT`ICy@*JNt-d%>oT>aQZ|hhF%LkwBh`8 z6Ls#IChR~8`)-7#WF$~QLo`O>ojh4Cabw9OjQsj#7>P-?Ucr#wvETMe2d&goyAom8 zz{TYjbUbiWE4?mkksKSDs|C<%ZH1kzV||ph|3yM|wR2Cp(~Y04-@BjJ2^{oudbW83 z9Dx4)=2m}Hh?O(W!{A~*LV}9IIQ7N*>({VYX&qYFR8`zUM&UXdm*Sgx=<#|2l@u~G zNJ1b4?xn|e$Pgg1mx_+;7ra=d^dp9%KK-lfpUanY{x6Qd+ryh2i#;J6h60>d=)}M7 z+}Es}R}080!zo3k3Y`20_~|}Q>`s)!-rsE#eZ~VVl(u5;O)l{tdYU{xsYDK$s&vYD z+)|nnB8HKNW#{U3W{I%^nkuvTXmXt%tg(%kp>`lc0M5Zwt6q#JEs2$+RlX$V&==s- z=}bk9A5!)Ue3NDPr2geOP8vs_r9RsMc_P)||;Rw%kc-z`(X@4=@bFZIsC0Mq)#jdhC=HiRb%T{WL8&6Rt{Yvk;S2x`k|SqZoE2D3v+D7Mm355{fj>K5u2KPaEr zwr7qfc$~~D9j^cv`Uw&lDl%n(H1u?j7ufM%ZBJy;U7o7X~^zp>%8>_fV;y7-|PUJT#Gd@!HB@C-;5 zgy{;y-(gSYe~h=EAV5>j`}AR6ovVFGPBj-Q5%9nx*SqO72zhsjGuHa{ZLhF=i+W^k z{8xevncmT4-3JtSfx1HDI4^XFm_R%Y0Ldm}D>@b0c5d)m17sanTwvN`_!)4PU``v1 zLHZ#%5V4QoQS?AJTO2UO`$Zx!lv-=Cu-{zlSD@!z+9}-KRpJY|+}~|fobI0bS0nVU zREfwZ?|O}u$Wi4hce~%FnZ3o4yU|oIAEZ(&WN?Q*9S5a6y9;bkd|hihF1;?_D4Lb3 z#XgtBWu0`VDr|BqK7Q=mEsuLk{Pcxgz^Z1#m2N|Px7sPsqQxN}Kri|sr?7-3z-VoeP%6RCG zd`I3b=+ld>7&YZ;4;^je1-5r>UiqN|XN2{z)x%M>@E!Gde%1utIC#}TSm&E^=MJ-6 zm$k9u++pjVV_5FKu*R~oa_7p13hNuq$if3E({_v=zXf6`n#nQ$kADs}-}&ATE8t3A z1b&U7TZz%3$Q@04$zDnfn~_baH~(^*M;tPe^7Kbvm0|&-62;bR2ijZ&sOh6 zMTq3X{hQ=!aUv+~3^@jYXT2a`&r zv*qVXI~;6n_jUJojts=<9J6&wB5I?UbK#j8x6UI@FgJDPZ{*`8tIoSNB|Z|(qnRot z@jnwd4SqIhGSgS2G#!6_d8J-`vN!3=P=OkmPJ@5MU;8&B$E7Z)m2tbu-flVg->po5PNtW%fN5fW;_5dVv({v zr>Lj+BBXt2Y|)TwtlDf+9MX%k87UomwT*1oE>{6(F3OhWu2 z(YqHZ2v1vkve3}5*}6UMW`3qRT4a9a<-U(=uXbIAsH-Mfla{!SWyXu{db`%UoJx;h zw~}&Kq36r+A+rJ5VYH$!xciMpZ!7!92t6RX*hb^+oRuhF!|Ij6uB_d%eN;RdU|K;A zms)|foBLU8K1Ano14fetW#Mm3_me>9Vv9wl7@zcQE167H)sqF`~jPU zzQObtYu>ik88GtTeXEd6B7cS9kJJJMkPH6WKLf~By=X$3#^rd#U+cjwqaO-Y?zim= zuhS6xEAejA^Hi3I+94AOZTPcHUU(qHg*QX?%=)PJR11N?Ky)GOvRbxsj>fFEH;j+H#@TXl z&i$OwE#n2u$bH*U|8{Y)*QYt#;=6AA@uZt2&kCQpB$ZD~$8W64_fS#-(lyR&9*@pF zyW|HIM~`0_e$`5yFhk{Zy4?lxpQ7>Z%&o@1IEba*#u{^T^;D3O&j|Y)A6e<9;8BbEJO1 z+}!q!1Fh+~$x#t!yRWzI0Q!8%twO9oa73+5WVw(_m$Bu&iSwFp7tE)))cueG++_|L zkdDEZ0+|y_8G9yDa%Am{7v?@ZY7qpo8^?W5Mj&caxkGaE!VeNh;+ocOGLNbbdy45! zQ`YF^DOXR3EyGpk=NzV!m8MpolFIYos?)ONJ*0acczeHL^44SO;KtLoI|WwzPI$U( zB3uY~LIoFd*#xh_Y4CYc^{4Mc+7*VHHX@ta?RyhF`;@wZ zgsx@V1~n$7LwfaZ;#mtn{COIpfw>F=&%GrtRKpZ#Jfkc+u-~q58We|a#ZhC-KzYCD zdtzzsSN*h8lGCo6<`D{;wZ_~HeE113K%bL|Ocl@qc<&4`fMR~Pwd(*UZ-Nyooq0oX z5j0^a#bOO-Pw`|_5FO>Gi;lqi^ z*^_{+3%YKzNahP;REn5WZA%fy5%jZ&`|MgbFXMh>*Ja!~kimvYK345oOiv`A!p{Tr z?GV@Z?FX|}7Q*_{l0IQ>vkOQZN10O)-wTIgE64&+;8rwj!B2>(jDBTR%Nz)}BH{zR z_dVV|N4PQ1MWadUW!1no{DgIl=b^6twegwnz3YAm`}J&qXU8*qL^x-cZ577#Q>K7$ zMi7&L;5jL^bq&=x+|(oGr|9!l1f-B{5u@740VVgB3+LQWeIsTWJUawm@OB0xr^6qN zEwucf-S3$7GNrj~=g~2Et2Q7OS?b@^pPG-uCEfU6dN{O-ym*+s?)~f&h(fzq?y}){ zCulN0(Rq9liyX}%$p9mZ?>iJ(8hlQmFDK!6yHdARVK)g*YH>>|($t@E-LYm@1+|J8 z%!2jPXOKfzJfEF6$02D=UJI6bRxZ{KCNg8tH=_$8h#SmQTkTeJxc(9Lyx;O{JQPZ) zwqk{OlG0Pv+S$jA{!O6&0J0RCJ1}#M#A9pOojE74mJd4m6N_9}d_>ga_s-*sHOl!u znT-jfK2B%BxOCjb_V!WRF8sxolkwM(C}CW-j=SR+6m0su`VpHWETw3^Bo9HGrXu)V z?R|h5=C}d0f-6Ms6!Fg8#mTRed-7(+w4ApCX$pD8gz4R2ORUmB0#n|NW}T3Wp0ZWR z&Wv@`Gsbqo1cQdt>jg5=aZOm#*Z$4kI0!2(sn@VUxsJ}!ejNRd%hnMZn`5}~5cj*+ z%B>&9hR>5j6fSp0MYkl~o=LQhX+fvCY`@5TSgC(9;O6#3uV|mwYg{v_!6E#<*yplW z>ZhTS|4AhL%!TdTpP%>IJk0%2&us3L{<8}2wSp1PX`|t*F684y2za|*@$Bd~ak0iM z!7;S=*#u2-+yH+~GtRjN$0z{k%`xHz&J1z2qL;@ZbJ3T9%|}vodhMic*8waH#l>v8 z0qwaMvV)(oMHT?RETF095&rmTkfXUt7D^aH#r-6F97~gL7B!L~a>Jz{B(zqk+t%OM z;#`^t16|gbHb`dHdeI@UuqOj$z82I&$IE7Qn0rf`BKN6DIK~cTL)o*~u%&B2t+bzU zpx9XTg*u1vLwR*KXgAhL6Tjob0r zoWn8?ZuWDJ+Vu%jPNpQLEo*OWGxdc%k2I8sMun>0z}; zPdpneE7e8&a)8e;{5%gs0)v{)uN1x>RB!=cQdPb{=uNPf$ ziXxi8*QiW|p{#P8zqK-jZauw0?P$b&3S^=K6x(p{c|#_rWQO^}2#I<*o0qBy#?S$U z(32YwLLpc67r0oNI;&B6!R;&edY3Jo^Wp()ix6Sa$8+_hscF!J(q9#S*KP#44ImMv z+=^ddUJoY3)P{X{Jx zEHv1{`q<6Ri$c&K(sFnT-Rx^^VN))Nc2d)2%M1xl(rHu;pdt92efCogapBB~*>2z) ztuhzSda)PzM-!|(j&niA11hJ8qzTk^=?1v=3t<^y8(^kp)bp#E0Tk9%aazU#x5MdT zJ~z5;hY$wZ`HXO14%1Ku+2x zo)oz|me=pXsNnX8Uq3%HkSBf$amuT9o0DAy>aV@EK+5lWBFXt8SNLg_w2|<8$~DL1 z5fH=>WL}G_({f^gAZAkLi~dPPse++7&o?7jL3`fxpkT?++p;pGDo*o%8>WU zn!aIIo>X{9wix~+yL0V_v&0_X16!Xg@)H^PhJZI^1s;n^1XWIelF!UpN}b{Umdm6z z-*aBmb26VeMwB``-(Vl9VNqAX?B86D_j{2m#SLv8eG0zgYcoW(HX^?(%wYo!I5W0d zC9|ADxonEzZY<{tpQZKW3)u9V4Y~^`8G`DqQU0V)nt)7h^Qf}*gF%J{4xbHIpkZ-K zp*9@6tXicjiJP?>y>t-GALL2Jrgm4+E(SNU3~TX2YO*YY++2UWL>Qj#j_wuc64@ga zUhEB=Syk^e_FQ$mmgIM_S6mK^d)AW;+0e})!Ln957@hU%y}-tkrrzn z3P(j<_`a!)9w3Ffd-PN2v3+Dq!USpyD@B}htcO0*)j#^3~^-d|p z*aewb>+(yc(&4-OP#JeNRi5QS4m6y@xQZc{71NT9_(ybxCRz+2M^bh_e7uTwwxJ^M z1kfT$M{5gJK`JleQ(MFGPd?QMVDy&i)`xc%)to=7@yD@=_ccpTvf@^E8o@i-OeFJ2 zK$SxgIGE#k-| zC3S3(R;()>CVrS;Sh$9c`u32J;+P%PLd2!BTnw)3;>Q8}#QOG6fvwgg%}9puj&yBg ztIB(~JRa3nOS#IODT#cWhwvqVeC(WCo&nva0OU?>+&zZsZT|AuL1X@_gkN^ki2%QS z(O;~<{HQtAq=MW>y&u1hp7*a>M>ddSvG8Ac1~EEZZw-6P>SWjTSfkVVb(TWeb(-1BeRoCdw9&LY6m-K6jt-G0g0-t|~Cp#RpVh`oIQ;^HlS8 z=ZSuhazo36tKd-hJ!oOV-vRbE zv~Bu#?rIDQKQi}RxUtGt1LhB7!nJ|eyVcQ#SSG(TuWI3OibftbFxMmD&Tmv`*KD+x z6W92u89&$(sW+ee#F=hYTi4ZQe%{h3oZ2C%VZGzl!Vn1yz4Uf)wOB z+@C}Aa}$*t3R7mT2{#`|)o6Uqzr-xxF#T>gzH_(0{vO83py_M75jQzF6XAQ%lF#-B zca+lUv{dqQ=tGt$x355OS>g*5+Q)B{Pp?I7z|Us8gO>gIU@Wa1{{u#Mudp%X`%79>e_wYZ;&zAUAchuRu6xXM4c(caIk5eYs zU3$U4UJWN%;R}N`^@pqF|6B78oIZ!q={bQpv7j8N9g2^6!Lns&i(xM z9M%XDkj!B_S8G4dq6jFIyZO=do}r6Xzrr})iK1GUH6MUx0yZ`lvuXcCTK z?kFkwH}YYxf3$tWbaXmGZTCZ&L5HNBi(h!#zp^CJHz}8`P%_T6frJ|TAa6Hs!D|u( zGC+}1Orl#0Qhugv*(LwhqxomhU5fdGRD(fiutC=X zKiuvkJHxlV_*c*4R-n(;>X^NU55VpQk5GRs$3qQZhQsMa7ke0HQ3$oavWQOHJ4%Gw zzNP>4)H`bINI9m41u()iS7h|`%06r_`=7kx^EY^k1*9 zwOVe^TX3Wn1rqr1D^VEdypk2UiBr5;D_{8)>6h=VJssRP_NYdaGdrn@>=VcI@5Fud zp1D)u_*ZgfSKfU-@tpqs*W8vO_9}bIgUwRt>foBPJ{wT8I;&Fa8WFxKQ8^XBC>24w zB3G*Kzc>0;hY!J~Y?O**?9`1CJ7oZN2&bMRMU-G9I-iY@grAOvqbv#Sq(kh<5Xs(a zJ>!oL`ZW?n+~olt;&pCIsinU~M*ZJ&p}2+s*r#J%ONYXtBqqTuC)404p)6F`YSXpk zA|N{(udWMI2A@%8StAR>9~$#2?o08uR8rY%U;iUB7VMS2=kS0}DY&DZX9up$4M=L2 z9l8+f^W17*V&YzxTFd9eR)gMC@&yiQ_?(`g@?Pf(i{CL%9hwdDb=mXPuCQ1~NC@+G zCX4=X)9hWr_S{7tqGFR}sJg!KS!@SCBf+rK)`b$tcL2!v>u&TELN*Lj+zr+8LL=6b zcwMMShc2ecDB6OHxDAJq(~V11Bd>c)XieRp^V6TZ-{+r~qbs}|0whpy&yZfhrstdp z!e+#{XWkz&O=|nvRafiPnzk!Sn?6J&+aFq_+$id2@Z}#k?OIT`GNazuMMJzy)|!`; z?%bb*Xx246d+AuRU%Pk3JxyiE?d+BZr+d(MF;I`C)8WY&KfYx*A#B5$P5%SVKyxwr z{sHXb3w}(LekjPS zn&adQ#nKuI5{?w@8ZTB`!zvW$;;@4kmpW4Xjsb6GziDfT3Mw^-HPu44&Or}Q7;39@0r9DY4U9(f*NxzG`!*M$!=cSkU zL0d@pU7@s>-~Z0M3sE4lcvD4Cl>)WgXsC!?-K-Ca8duP%i|cJDFN558@)=#xeuiWD z{4sJbJn~Yb@vy^FI+_i7991DO(rJ~ivL`!}O{*xjbR+(qSar7>9|SS(u_a>r(#-75 zUB_PzS}K@MFgA`1)`y5c4c9Oj@;>%7NXCfW$qOx?(>a@64%9^G)SazMcx@2Ua#-;v z@3&a}ElbhUM1f`_*!@>M4avBPCQqzERsJ|tT~0XRYR7OfLRSFCg$vO%{C$G+^`;SO z@!ikeDFZ_q(ZuUyk4rsaFWN-;{vYYtDP0%JV>2hGTN4vmx3IzrC+9mwej+1**35Y+ zuIDQZ>4jrTcd()HXsk(~KzaW@!lFWlylud^Z&YpGZX-FK#o57ivE*se;|YGnU`#E? z_e)L-Xv8iaq8`51`i?QZA45rxZqDa2!UDB|qPp%W{cm2Q=}XV79T2=V=wwe6I_HU6 zzhZ}6mHG3ZwBZq~_8xr3n86U&_5rg@*ek5rjd3W=w1i4i@mgd>oO~Evu@!}e#9qhF zSCH<%K1!jh0x2=s9bc4xj}T~0hmNGa+LD${SZZ;OX%=?qyz1=Be7{%Mwnm-p1lwlN z&V2J&3gMGSN>d^~RL zfJkBN;dX%&@c8%RuxfUev6NIw=*j|1GvcWr_@tjPn zy@{PZJQ()6jT@F?4$ww_C3t?T`4YNZ*H2~>3Pm+Tz7<~~R&72HY(K7}Pkafg{rI

E5M~nkXV=?Kcx`E>fWBHvl-Ws1{fn`lp5nr?g$a6<-#QNXgk>1 zkqDl&#JZ(V2Clgc4_L$>laz<`DjF~aaT3XU)V-0Kz1O~IY%!laFK*|kX0(}Y`sKMf zWmLHhk-Nfx{1R14KOh_3RH^%5$R#NKQv0=nG3J9sJZW}mvntNVQV;(S-55LvnE==J z+1DZU@84y+m0=X6chZ#U*9=Z5?7&(C1?-Uy5_9=WUT#=h8?7oA8WLFku1~Obv1i*O zAgyz#cFHnbxuc$RJnxt&HryZoOd(57o2ztR45!pCS9{FbqWG-Itq-;>7*K}atv6=G zV)R5AOwDq3xc*`CuC<#n7A3Nwipe=AtpfVto-kA|4D@s@wp5#c>WrcNo?S|JB6puP zdKg;i4+#Kvz}wwA3L;mk#6wATP|db)Iryb(n zO5Se%{j%C}f)bewW@*GG*9(_Y6OSlY|7+N)0uQUpqg4zv;kEf0S-# z$}l0p)V3dlp*Y$;Yuw_`QE&uRAv-=cF zMw3Htvkx@+ZlP-rO4Gxmc|m^EO&&j0nTw+1)PDW42y|hgY)5|dA}ca(|6KbQgJ_Mm zgSVHUR1YSVjl!oSJ{vqqo72b(_T9a$ccK8r<(-yMtHTK2&eX^E%B}m?a7^GF;fdWR zqrZiKAb-dU3KGA!2n9%d_W*v&SsKqlwH`lp!{;YGRpFR6$kQhrD4dj1$Lko^;U*gx-Ga*up9%cjxSX%PK{wfwo8r1Z{Y4M?gM`b2hT;0z z`D%=XY{1aXv_fv=y{{+j&2@lTQ9wC$i~AHAvRQ=wsP+oQZG5^g`(|zHoyb(67WicT zHZT3+jhvbnrP&!?y2aypG7OPa7W5m?8nLoNqzRcwN;lfi3`XLt&L=(9ve5drS1z+Y z8+q(rNN2y1(S?``txaC?fo^S&fsqP?0@*;y(MZ`P*e8f1EYG z@5d2sv03Gp{@Pk>IYaAJ3uJ{)DeCG2J6J4ff!*Ur%nu6J?J#%W zd8!yaO5R?y#7CrB= zMoEs6WT#;%Y7%TCfBWIV_~=Rgz2_6kkA_rv57m~_$`KSIAFzw}fUO{MhykrLna667 z>y6_M_5>XtL#*D5g5u&3c~6@o+x@X~0@icEG2QB|Fois%{mdm7cwbL-p?UOMD$569 zZY}>@Z2{MYemmQ&RJ+N7@-u^>b9KdWFX5cg@0=`pSs`(?Mi~tAY(XLW3Y;*(__Pk zZL66h$HwjvoLy8QxYRmhy~GL?^>}}=KT{(78V0|x&OcLRZlCJoIDq|RnzDb0)I|;l z1c^DtAq&2Qp^!E;m2 zu)LNm4q79B!Ixd@=bxYq_#D^X(M!YE8Q z-nfg}Ubnbh*^GquqTEHK0LD}A|n8GSGZCOcVn9kI4=vP=sk;R&1JYW3Lf^JAlPQ2z5AU(p2b^Sd5?a$eDH7 zvh8N`(_0@E#ZJ!jrf1y-SsTlzccNRr*#36D2deF~SST_dM;_Sfc&Vxo34K?g=MRIwoOb*-{rOdJQHRJTiM5%UUf;iEb~C8J5V!Iu&vVP$BwB zQ?g07(|BDCi$B-_Xr)FR&L3kcSsYq7RHM(?RW2FzE2G2L&Aw+PUjwMT7}!N{l7EU_ zpNyJUF}OHjL+5_X4PK3#w;4?^Gof=m1x;G^Mo>(|qa+?yU=Xu-Ok_10cOc8!F^OAg zeKx4EpEn<6P!{S6B{4eB2V*wzCf>GwXPYkHMD+D15^lZgZ&(jYj&dtBf4&2Ib2}I`Q%ZpR zHMT@v^t$>>@bWKF?5b$v`5WCN2z_aVLx&&dz*N0$rStt&g5lSU9@H}hfnPNmv`Hsz zV+{899XuuzcP)SP{j>w^TQy(nn9bL6Nj-4Ge(Co5mR%IaI%-R2DRo_3cs4YGZhVD&Es z+qxTM8n@erV_Exd`?_axlqCja+P|ipq0BLDK!k+#+RCJvA)}Ac=o5a}mJo7iD(-q4TF=LBL$-V`Ni+GNL*|}kk_D!K zh@_m+lv`z<4zpOD)G+_D1NFkX%`R&^3+R&~sPqWV7bPHU_ex-1_-+h{f6>2zGz|AuEnnVjDPPXYhKrxV~etD;x z4VgvYo2`E+QG!7EN?wauHJhs0+ut}epMI`hwPIWg(z3>L#76F*vZELs-n@8gWOHo7 zf%p+!3inB!?%s7YqrK;N`mAqx@;SwErV-?dtEWJ{xrWdyGZl5`0?xieN^aJatQYrD zwVS!84$b$Xy9LetaFZFko1eH!{Ky)~RstTj5!eDP{w%uNszK)QN4L4qC}|h5QsHnE zQO4!fhuhr)EV-|xO?2{U!|;^$>wim-SOGx5pq%R$98DRK;+&C~lCUk$x!GzB`HLg% zUc)&alinUa88R1}d0EJeEpkOO8DlA8(GflZ+fP-PX!^m zEK|1!m;w6m%A-vcjfOL}aU}8Qq*?XL8}Y?#ZIWPNq(UOk@6|@Zqb8 zy@uddCtrG4k|BwLAzq(W8|Vt9+p4sWj)W=o+Aavc26d7uj{tdxa4 zv$b|s%=5(r9D>z55Dnt@21UW2W9X~UjkRgp6jD{TQ+D?BmWqgG!_k#D=7e0tMG>A7maTbqR9pNqAd>dy9 z`eIau<7*~w$4N)pK$eTtG`2V2Ym=!@pIj3+vbx8R5SJ)}I@cVfef@b2ZZKW6@m7+f z2;2IAun>{K-&sQJp~VfhTh3g8pc1=J&zmPVowE}&!2l$}kjq@DSES_(SSC0EaM^l0 zw<&i-dXpHZChvwYQ5%py&0>e!v08K6gVIIAl2;G8OpotdD4Z>~(Zr2MrfZcIGdT{U zO#RS}C6||K)Dc7}S`$6_iQ2^N5r_*s=k0@llEN~FSbwnr^8~*8td%6 zTR&O69r1cDD{~Yb^berkt>Ui?#lLI^51la-uS z1R#`6nZ;sR@qN@9_Mc>si>R-*)n{w*qzzgen{o z^GKS8ts^=d-x)@Tn0%$E?)$fyE_Vz(GVDdn*Byf;F#MKjZA?CZ zou#$Lsf$^tcKUYHK9lJMEGi8}$lhc8M6cbi$Uwz%lwj;0NLy`vPWI52MjoY*T0|!# z9ywh14z^YjZ$8vR#t2?y2XO1+jGUHYZL#%M`o)`#627x&FlEDlUI36{Oeh0?l zWIh0N!_VtjTXmJH)Q}J7mWQwMF}O9`8@bh*$thd+4=S0$ zHWr7`R&2smhH!{s^!u z$F*XM3H?Vh4yXqM3dr`T@K{O7Kvf9XSU`15Um1eS0=HA-;=1E#iLKN6ADKNx7V8=c zLq^0oE0o&WvXHJ-76$Juq$-Soz{2^Mfo2-hw-!LbOL}G*4(Q?4Covr(yFVcsb<@W$ zUSjOb%?sYgsxdxH#Ai%@Tt#;gy=fM{P)WG6dUFCFul-7 zs3$Jz&NGPMW-{w;hD@N+X)pCGpF(q2aLumV7Ze1Xp^Ms3Y_HT_6dLIjMsKZ#EVT1pk}}WX&^e zD90e%4XDJ33h8P4bdH~>tVv*L8n}5?FeMDTXnuGAV7#?Ia7H9Dov=U$T3D@5MmCAp*AA z<1jjX{*L+V+WS$h@LxCw@JhLEMtE-Pol>Dwbm1|L-XgteVf8RGxCAXf` z3k^nXjHTHIC77OBJ{(_(g*CfKNqDc?=%@;zXZibMW~Wxb^ORG(>6*2`t1r9AGegq^ z%j4{Q`dNqTvsMKOJ(AHKD22%5kWxVmoioHTbrIf9ocO=t@|+EDl@B^#=%`wB71YNT z%_r_zc5S?aG&ZxR6Sv)s`x>yd`LwZ%>JvYFU#1XEV~yOVI>|p|Zzmy|9mLoXxn?^h zx)gNY_QQsKIZ_Ld*Ey|YU|V}W!hRo3ue4wJ;7`AMWaec_(zl3QN{IG-^`$5FVDtQO z2Rp>{?3*p1sx-EuTs63W(-9hYwD=X`m8Bd=!*elP=Mwj)6wkc42_ULYb{jK1Xsi*> z-D=95xsg%6+U{qY)+g#*PZpaR0}0@!)6D+8U&B>ErT@=$o2uwpO z1yD+A1d{yTXI*FPdCRI3wwLAB-I4ds%H3QTA#&itR|KeoXTBezq>&x?+|WGrky*U& zE|lH}?Zr~0JcRQ}#@^3=uj{&` zK|v|Px(bT=t)2>tG}Z6ARapXb=%KRRDZXtA^JB=B)q~bg(8??4^gpMZ{KeA#Z$&~e zYWzIS!t}OKd~=`@O*Pxbkx1##rx(8w*}0#^_z&1FF%PgTsKQDt{ws7 zf3Cy#kJ<2(-)8fFi`tn|_zba}PdN|~R7|64jQe%Jsn0Q0Qpn&NiF;DyPwsW>|5%?e zc_ROhy6C_6q4p7DI0P~UzHZHuEq79olfI=dJf?jJlMH4>XU_2F`a(Ohfse$gzpr~K zZ**fXche%yzlxZ}!Wg4~8q}n6x3r-|9>aC{3??tiInGC&D}8)=MVhC)7V#3JL~|D?w*QGL2(qWds7o2@in0LhsNkw=aeaD9XLxPg}iTb9r|C4!$JB?FAuZn@ZK z0Ql_ZAyV7`Qqr2;s>12IJR(VNSk$3P1%L8b9ao$R{`hwOT`&LmP^}EOJL&kXc8IJ> zppbvdsLb(DKncHF8ET`=F*}KdWP#+90a)zHeQ8c7m`Je37}RM*e{@wGecAECt5{@* zvKwi?OOA!4Qaxf>_wfAj(HH3sr`o~zH~Z}GX(FYmW0CIV4Ae_<`zQzIh5%X)t}X~t zVw8ZfF)ACPnbdA`zx1i$=BWO?@~+<(Tfj9{_ig&@}c zzqTFaF22{O8Reo5x1Ia9jWclmEMK|Nmdp z1Oc%N4=KL;{}LMcdx`r${1r1N02);heB$}Pj>115z!e9=N(6No+g}W}|Ce^rqyUZj zs*rl|sGb`bp_T)C-k)*||A%d(6EXz&Ht;TJvJFOasaWpC;7$O_n`yjAjsXTKed zrT=HF#kz2Tp){?lQ2(!929gJD{dv;e!zqB7`bD?RE3ccqz(-T0jxV#vQ1oX8Zml8% z+$ZnHsQW*TEjINyuf8xGQULi#EF2l6}h|KHiL^qff}Gp zyhT^$pXnb~YXW0bz7D3i0tSMX zXG6mQx~@|)d@@Q50XN9`!UdtqR{=0CG&;~N2lCkOu-lrh26xtD`+>h&;f({ledXP$ zmPOO%%A}r{7!2GK9}L_xxD3kr&pq9w0e2In6Ejo<-OGB5&fJ~#mjfmEbx4;6=yUJM zpR)hfbaCC9r{U2)d9(UWwWW#qagM6P3+jf?+G!^e zqkudDI}k#3fS5<)kI-xL)i(+9kJO>hTL~ccy!HL+ALadk7X-TrdC*Aaf241GczpWP zmGIb}?iLuo>V9X+qe+W>KIkSKD`5rkUG-;5>hsALkI(00jC84h9^eNy zp#yp~tSvDO_;kl(uPy~apZwda=74v3nltI(f-gyY1Rxy2b7n=aA3j?Zv4fE~eCi;a zvPEsS&-938kd_zfaAjPVZ6Do(+8pEv-25s2$4feI(zfz?he<}-bYM9UBm{DhQ#v62 z+MD9~kEW_mAdn$D?8IL^qg%{HGVM1AcE4-Nm`ERg>=HhZV`&<=ilJOsMT~mCx>ChK zYa#kEp3Z^0%;4+T0r@60xREfuL^p0;pjW!^j9ZBvZFG44S}_0viUb%MtuMqDkep|2HB4g%gc1acG{X;1HHa2rPeR1ls*SZSY!Syiw87En8k&@F6HUC4MNweD&okb4Cr69<;YxehVK5HR`mtdE7K4WMXaL~i9}BJPujzim9hUjx~KO7N$8p>{F~en0<6h~82s8DRBmUK@S3FN&CTZu-~T+2B4DwC z(8C}kQwl_FL`LC8?w;xtEMU}wP>}ZLznbTIX-%|Vd=t0O_7|gx+U#%v3`3uqKiC#< zq(cJGQORvF@r%8an3;;?dkO0J>*3hbQ!ApDz}BSOwWRWNI$UX_=yaV#F{@sj@l!1Koc>5OAqWx_ZG*@_=?V~7zKC_mxzh!sTZ{b@T7Z7} zi@3V+(X3cz@hgFMwYutBh=vl=&*~}KjX@9)jh^Awx$AjZ;~UMqp`*iH0=%|~jyYm9 z0T@}S%Vsyg?|l;GsQ1q*qn8t}CxA*G;760iqEGaYLS5g~`{(>M*nYTmu3GpRvp+ig zZG)ngb(sEfw|vhNyaqGY|!U(F5FC za{ykAe=#BIscNRli;E`qJZ5=HzdpGjQ(x|@9v$(dC`!ji3vy{{U0mhRWPGE!ED~j& zbXb)e4R;QHFvmfho7Gagm!)CUELg2O(WUaSwhB)P0fR9nT_?Ad8_x@}0G#CU_iBrp z2He$XPMVkvwkT1y@VueIo5CsoAN12HA76HHd0#v@gpqizIo!4TU z+f@%6cL`{GG#9RG>LDAC3u)kll~i&P_0}3a{9AWv)3I+kgjWM{ z9e<`l+d388|2T&+Fp1kpJM2t8E{y7W17-XLkJJW+ySC1Lbfh8muPM%%TP+JIcFo7q z1M~hRJED9@Z_8C=S2I*)*-()6s5D(YX5~}xRxJ*n7^^2lQ@=h=AZ<=IR?3c1I{Ui* zp})B<_(@eY;G9uyl6cBH*=DTt_7rQ)H2Y^K18O6dd+dzF^9Le?nufQV;JJtk(fQhW z#YEjRTjn*zrXN;A#0S6NL61w9OHstNcAAl5FZu}d1pI^Rm%*-Q_WYiWyAONlZ!v&{ z87iOQ-@dR2LHcmL2#E{!X{rWxjrAs}zV9Q)+jM#p@jK6#Na-L}2A9b&~ z#rucd>BmE)ag~w&jNzUcvDFvWxFXB`AI81|s;Omr9}yKrK~zAcDX4T1q<0ITARy8^ ziV8^YJpoZrq)1mOk&d*0gc_Q(NR4zMK!DI==%FY4hx@L+`+x7<-+ODZScd@T%$eD< z_xJ6Y*&o*6`I*Zb$@nw)F+#v|9_F*}sUGKTo@1Q%$XoaIO2=K%@lRa9#=bDPk7cA` zORF#Vn>{K^#SSP1+e-7qp3aU@zZ(L_Ty7;F4^+QB-NN5U$SF|1Fuj%s# zM17e|aew32|Ne?Gf8Odp{dh}3A?quv2be_qLHb8$2i)b$4iN?G49ORVZ>fXqi65Cv!lMg5EBVK( zQk@LUc|gT3?RNM1Kt2yBF+tZNtdS(yQ|W4DGgz416u~N&G|ndHqtg~IkcT(={P5}r zaeJd8pBsTxY+`|PD;o+6dd0H;rCa{M+IKHoocmhoG93*{F}DJB2hOS5w~PYOFOJYu zfn0tgs3Cdxpm3$m=Z8SmfJpuMFu){GFk(dtvR4P9etG)8H{>*_z}Grdpfr^FMN#WZ z`+IAVSrqzhpy1>?P|o+c#CIT)fEalI3KpB}5b;a`*3&@b^`O-1eoHtrlfXigDwTYo z!DcKKkqQ#ssphhu0>WS^QxfgvU3Jm?do%Q)? zpauTLVN^OfMaW#fHCd+IeKdkq=E-bV@=k@<%49T;_BA}q<5NPWRXV7P^1d7Spn&Tg zP_p1MD4ABYs@N$G3N894fpX{GKrr_yaiL$tb>_Y|5NF7I+#D(b>Ug5O@sCRnXKsfw z@IAql*+)ycK08oH$S^2=e1=bt-&fpT_Y}<>DD3;K(sh=~US&IuPd^?MLzG!v4FJ(< zM&^GYlc6S%Fv49Vl`xR59M?dPplwN_6?h?DapTWXF z0bib8!I#y?Tv};&cF4-ze)s`_H|z?vU5B6Y$M$AbAC!gnYVuy4>YgPC2TT-J9f%)% z4j1|PFjJMm)M|u=a@uviCvzIDu#d3n-39Goy?jH`8 zO)4qrT+ovlK-?O>tB?Clr|npKH0uiJdk)R6+cf{%Tx zbz74(gqm4?;aJ0 zo&S37%ylqt#jjNSne_O3XGTp;`R2^kXiz)#=pEyp)5wd**cD)r_2ina8 zmk@t2RWpPwTMlMsr;W|fm&}e7IpdkmL@p;G`zwxP)z1cp$10o~hf!VqdHSp<$-B{4 z-smB&X(SmWu<|zsQE4U|2pVm~_eVt-G$;e9ohQv?F`H*^#c%gys%4KVa<*L0k9s6r z?LjaKEWPpgaAUaWkN0~sJ`R?G2d1%DV%7d%Oykm}E}y&!&Ml}0|6nM5@P7Cy zC}+8GyN0g)QPb`#7jQ~+B7TTE-{n!@$<~izc_THxgGpZ|w~UUPdHjrTeQ=R8rHDSZ z90)tv_b18w`&0s@oIRkHvLm`USnKag+5}HlJt%a||CGBFx=WDY4LEhjVH$GB=EYy9 z>fc@mhFqd(7Rq<^Bn)45U+6ooorHboV2t#zKk(pgWk9f{oey5k0jz|f32)z%@kT?U z&77rPK!o4xvWI8)>pB_BmXlH-Xsi=3`o^d5_sjPEfw<9tR)V1E6HqH}#|zZxe3)+_ z%=0kg^?~BcgoF9#hnZJ^Tus$|1Ag>}8!E_ zB-4Q-Y5i~`dknQdel779WiW+6!2jq)8}47?;D^ZZ(Ss0lK><=-_H7iD*VTJT%POt2 zJj?Tqruj2Hw|43!Ik&lY5ez5G+m$VJeYch$x#pCEP!(HjiF!W03gYPzXlo4DphIfZ zA;uXnvph`B#viVD&n9a9uIzzw(xVEe;9Hw*aeVPg!uh^hj(0&-@~INEd@u1B{>)WX z>mwVW$SiUD=TJ$znov!~!6F&OAtQ=4IWQuY$Ji@>)dKk20s1w!sBZ+mMr^s~os2Eb z(}xc%)=W$eX&(r9@fd!|@PL_=KG(e<;gHnz;(WtdE)6$>)|R(SnnvXg0?9I-8~n=b zoLNpl%m*>abUZ=Gd|)0#ihUFqT`R92y92je=Cjg zrOcGFJNEY|Rgvn<-9L7rse68=fBjzR7{wv(hZBNB>Ku)egM``?8zw=M?<+2;9j{!@ zsXl7zAT2l2J?&0Ysv=9dOgWi&{QMzYa0gfn{EBMqH|H+Q*Csq_+@DLSG=Lz;|Ent!BKrY8CGmO#?OTRZBJb zaMdvSdnM*XvC}hPLWK$%39oF+jR!MTQ|>?WA8eDRT~8h+gtSeZwaR~1`%WliowP28 zfB%fv#}O&FJ-eJ$pLRFm&MC+2hGfVJ zj8DXG*aby&JO|ZVRUl<>h9CQS+@#K%@5harGZ9v9t0T=8(^PdBQmlY)ghBSh%)7j! z3O@S0wMIWwaT-(flKvCyDA(Q1HsUv=w`fIgvGy!00Ua&qE}FxT_v_*R=(X+;MRbZ> z<^kXYNMTN!X8?$BT+#tSik9Kpv%e~Fy)L8@B>C#@L>iChjGc|l2JG?X6(6>wx^>tW zMK%HLc!yXsm`=j_XKiJ4=oh_z4$40sc>3`1b3$gaN6S8{Ukj|ibXw;6$FF+78QwoG zw)g@;ogvjy^s&}X<8o5z@%(c3GhVVvBKHy(pabS@`&U;Q^@`g5_1544r;~tt3%c>> z#9!u)BKL8G*V3uhc;R&i?X@T9({0foIu>c&nP?T-ef!#M{Io`M1TX~$EfiuXZx4#(^3NbH-`hg`{(Kn;C5h% zg42lcc!lFv6T^&KtuG=FfU&9W7q3kZhbQX! zBuP4F=oW~q;`*>tSiZrjRp?;8>ocpaHUXEdrVIY>1uQRqCe6H^i7l=1A!Xe+-w3h8 zc5kNfXV$QlyG%9lp8 zo;pOYxDI}+P3M{}-oF}$Rq^Z_f|wR*U>i-eFIheF-i^EEGO^0=p~xb9()q49L7M-~ z*{ko8414aS*v)4emJZeoe2sPexqjrs7q6iF5rm4xQ{SBig)=gG{P#XDq-Zqv`IlNd z@EFU&ru3$jm~JM=@BL726U7Ty>G+3_JP`ckdlDwlSexq@{lb2e8$Y3B>-lYignEJr zv~<~+tXsX4I~6UYa?QJZ-F4Nc*y@F85LHts+gAH6RAsQPS)3cM*?R0%<|q9#HDA0p z=_|L;3_S1nO3rm9E1B^`t1X)aE0cAWx_gK+x-}8)9=bv!bVS}HuD6f`GyY|StuPc; z!`+xI;F+|0rv}>^bB&R^jj3Q%klBMJ+AQ|3<0(+5KV8K$2UI3SJo{lg z*dg|X8fmw^eAzXfqdaM2y44uXr>eGJf_;!PXwvF#y#E5WUAU8dU$8e*^|H_uR5drj z+pM_lr_q2J(a|=|XP?lM`bg;wt4vvkVWn>Tez9@s$|neui|qDG2g|s@x2h|9Qa=+a z7K7Hpu;Q&nB;#cB_`l=pv>%FO`F9Y9FUKUb8De%f`(kV| zpGhqVcR{dU^vkibIA#gQ_Op>(gG4|02+ak;T z-SVFZy@)Hc6jM0Cfw^9p7rhN9~#($RO)e^ zvcsR35r}j7+3w?b@9P39qM_HhXZq92FL#oE>{_BOa2q9VUd)SavSOXz2)eJC5$!yh zX#F%+j&)b8cJHPh_p7`pY1OYMU+L9oC>)P{q8!boe#Q6M`ap=g^ht)MDzS$(_l0e= z;V*RdAHRFZ2oq_Iux}CNDHZ8cKYsja&+K7{ZX2#N!nT^#Z{=%ZPp*d{YCc?st`!42 z>68)vvq!P5<~=rwvFEF=J>-FE^(D4%`>y56mw9=h?<)*y$u{xnUI=fTkSVK;q@r(~ zzF^vJ_y?}?{#prd_)nUiyr=GOm?R=NY0gcf`xmC}Chj-Rir94fTo7~3g2P_Hp1E`cM%qUN-HcUZ}_nezb z73mXM=4ys(OxxGtH?sQh(RYzP5m85M%Z7xCH}(^*M1?K%JZM3ibN0rQA7y51e7Ikx zzW9hWF?qkxECGI0b8WjYP#2}lkZIHZa7M(I=BmeNL}R|?PZrbV0>2wB5hqg+O;HZC z56tkJ?e}ci1;3`gv+uS|N|N<#HDR=4me8>KY*h8$&5>MqMfEh+xppdKv_CEE6Vw;Z z(8-DO6?dQddMFmb3VF{F(|?P`@r;ZI|NJl_)Gx;bhHRB8eUQ4qgte8JbC@OAF!VRP zyJ?S2sCZrsN^<^bMVBV2GrsAYoyqEAbe6Da{WDOUenpj3LZ@XwKlbt09|DJZY36i5 zzX@D3SLWhLC8PE+T&&AdgoaA~4@le2px_Y~`Ok5;HV{RNbjj8C#~h^Uvsam(vdOrc zjXm$yG#&_^VN*v&dT>K;SIy+`vJ;oH_L_+(v`Or>+!3T_@(HwY`O@9SLQX&4DcL2g z@45-A<=_oD-bgGIfAwA(e+uD+c%w8%@PXnBUAUI3A4s~LIybpgdLf^`5kCCZY1C21 zDGhA+6%4q=bYelzAB8ld%Vf)cpW3vZx!2r$yU%y?(JcMf;Tb(A^USFY@R=B=RlZIe zCj6^l$Pw2gnSGm}K}Ytv+qW;O+SEDEPFM{3UaRjRl`3OjZWX4IOrc4sj92d$kx%)? zKUGkccWg7Cy6x=`P020@A`E^Y9l|l*?MENm)**8UG181DTdxwiYhKrzun!C5Clnhr zCxG4psx{CtotUXL81>^nrG*tMsJ|a^t0-FG_$zEY zJtz9)WBteLqLef`6{_Ntn<$#Nr=LbC=^0ysAf_Chwz2W?{w>F28$M&?PI`Vt z(k8zfZEG$zjYyG&OAQscK(#j$>xOMcxA#hhO7rlOM_lGM1kB1okthuY?ziFfSC>(t z_HI#67DfZrZ~&Z1=DIEau<_QbbQ50Bi7&0l$;UkK-=^YXt zX(!kYwr_|UaNy*=(2NOJd}^sXUSwM84NF3&Vwi{w1=7ATW|>T%A7&^^Q@=bz{qiE!BD6NBq+=8*@#={LK@RP@wAj_v^}*ZI zk!__M>uILC@3lVU0V7#6lk-a8_d>`P=?m?_z6!NoE9qy2?!xOx?ysD6;!)do+E+A# zNfO8UbeD3jnSZjrI_ikieU7Rvu@)l#8Gs$0zuQe-7S?`6xEGK34^5Y>s&_Hpjw}^ z!sFrDtldKD5}i7wxWzicuuXE>s5{j|VU8)OwNRO!K$!x4Og35L0$EuS;a<$Ra7J#n z)%JM8O&1b*8fOqf+_T)gyLJvwn}GY)D~Rq-o5oet)ELl|U8GQXuglsE+3-B(h+UMF zaqP>c<}%lWR_=gPuiXxkx%5`tM%s~loCD+jS4!yuIQjVwk4w((Fm`gWBK+z71P2$- zJZ2r`46R49&}We{_KzkP?i($ZZ-ipiq0(v4P>Ny}Ey4wJ#n-uAE=g`<;e;38sFQH* zl+&*_<>l4-H+fRtw&8R11W?|CqcO(LXwloV=G?FS=}2FLZLszOyyo%y*ThGb%B3m< zB$SuWQ=f02l*KJAyujBvD3P<(9MzC^RYr{)eWgr{o(ou%!Js$p z5Lc!qFT?l*3l!NuFr;#h@ILPlNvAo1LQv8H&O(uZiWCZ7pZ9bNwG~jwuaV(Bv|Oq5 z-jp{U`M=GDeq*yB~cZ)VV_?FH;!jDmh2+BEQ z-Sz?sddu*I@A(7=jV6$da{WQO1z15ucrvtNRNAIZ_520Qn4hQhNMDOSLp+u*m%naD z$^$DP!@}heaRRgO`uk5(8fmFQbcUO^2#>-$LQU0K3Gec5+UZ0LhSaU>Y`wS%PctfG zp%$@$CP#5#P-6IXA(<{hkcT#Wq$6@$e2UL1$lYn@E9xh^8I*lL&;f1&Rk&H?elh@1|1aNUXtRY@IpZITx0Of&ORHMU63ooJxtY`QP6JXd!Ub~P6{ zEo4`{OI9hh*K*!d)H-F#T71n*XJ1c+>SzBuu@um{g^zXpB<4|t+8kMUgu!j(MRJQP z{VH3jC-&TBH>ouLy?#&mhw}W=G}XpPcF(lVZ81;zVFzPb$?b909`9zRTWv701Y<;_ zy!c-S75IZFp!jaP$XQbE|4+ve_9!qe!GxPoHS~NQt#rgHU0AI4=I+%p&+d3HotEKKl#@AK$(~(? zm?OIV?}Q@E+3ot#7Ya%o`E!q@yRTHp=k}*eS1Ri|Nu}{3q-*sN><;TG&GNSEW*MeO zp1yRV?{XIxCyj{!LvYd`72=h|2MuSN`xRIZ$~hL^_0KKz@gPnl4KB2=-=8&EiyWSS z*W#i~j+kH`cv>YhFy&!Z`x~1X_o@UuDjO4*YGFQ^@E1!4@G|L1M=stZ=I|VObC2%D zX6brDW1kDwnqJO&*>}3p5Ba=%ry`1UMG1+gwn!&&t|)~^UC`>^chFKYVa;P)vP{QE zE`Y>oRkk(JB%*?>u4gW-VB#~ey^6KmHW_KTIQEulX?8)0=|$07bbrI>Y==TWX@u*a z&iufuWA7+Z?5X!v2mkW$muQmqaI6pEQ6}jwi;kTmpNCVCcCm|Fv-Rs8e2#8ExEin` zd-Kav+;*=$zV$m}ap>$NJ1-0OUZB&ocO4Ac+N@_n7%-BZ?``z95oeAW%eeb+Rp3Z%OgSRna zrGTyu>eaBV)IPcEqS zi*8M@wozp-nKX1y`7^@s^7)m@9QH=}-JLx%FCxK1Q%H1)rBtj0Bezm_Ol~$Yy_ceb zymZy<%SH-HLi&EAJ!e(R&|5J$bGR}T-*4+zW*uX~s&$H;p)*`L-=F2{p2P*Btusnk zZ})34r+>}@(Q$<|-6N&Mw~@0~5CN52gPl|0N`_j7qURpdbi^H=$hj7&L3Q-c`1xlb z-O8e*9CzfMb^je*o2AQ3!lFV;tzlJWu;$(yv*mOF*~va_h^<>|Vf6FQKkcZ~o@aXVJUa3x0eM^DXHlU& zu{1e(X)1yiY$!1`FTLbaWffYY zQsEk0fs&-AIU>ZWmv8E))*V_4(*C_7r{#%hiM?z}zMEQx@QQ9>d(PiBc)g0!bJShm z=+P^No+mMHqLl}G@W&BG_2%|@&7CU0V8{2H<5;0ns&#c+@wAxq(S=A2**b%>u4r0x z_(JebJhQA;1zn!P~=QM17vcl z!J6m9_UwHvhRSu0HHLyb1yxiMM3=DbOz1k?TvbK9IkDcWp_@0C4@FPz&3?4c67AH{ zj#$t1+giy^4i0B$g25-kcQ*Q-BIlH6#WYu*zW#h|--q?b1gk{K?lSHX&#=G5#c+Ha zGhql~T%M-xl3!;Wk;?9GG-mI`ES?;$JoqDh%jDG3(kGvMp6v5v=yQ*Uv8Hl&8h452 z5joJvi?wsic3~$-JM{J?$grt5n)-ebd7QOi2y)~K{GNGkm>^?i0eRF%3 ztn1ojA8j-JEftx3W4W;1-$q4c#OnmKF0%*Cd<9rNemt@XVf8poRjz}5LZz%}v9+%& z?}OTS{tL<-qU1dHy2uiX>$+F$V4v>Xepr(MpFNiRSbd=TLO1PZAXlx~lIV+`Jcxva z8?B))A){G3qu^pUt&gH9&6Fck>KvTzhd2?GCF;vOqXTo^W-X8|&`OzoD^zTT$>{1uIE-mG9``AEP}8`u)MVQU;ea<@)uUsCp*+X-i>i1!H|yG_j?8{Y zAKmbB>X0|i_#@bC=sCQHTPKsYT;T&*VIJd+%<#I=+N;j)*jJ$&--$R)1py`Lq-&E8pazbH07BSqA3Yr@!`MQ&YuNnthL@cobwYo-B>|AaSKO0Hzj+sN;{g;@~$5!n99>iQ&g zLw@cpDn4Sm{P&AnJ$R{r&sezM-p0FBsdsN4T!SnO`Uz~>673Owg@tO&YwXA+Mw|Xj z-!dAgrLTX$1x(AU6>wrfg~ zL?jK_%$LP8o_kO!mvX79OT+}ZK2G>w_=8PK2Oe$UH@WQTlXgE zem=okMSJa}K_!VXb6*x%CWN`5C$(TLp?TImG99fUF|uZ0kIf;=+9iZPk3{t)hwB-O zB<~GwKsD=Rak*{|-98m$H3sMo#-7J0-%~tol-Q=m|9;VL3tMVRUTR3}$-)|6>o0#4 z5#ZY%qU8m&hG~tmu#>y|_B-j=OjHU-RG#S6s%MH;HM zO9o0#G)3w+EkvoP>eB;fGyk-lz&EDjC@xb`RVw7BsT|2KvUAloO~j$jJUOOKQCwWP z$gSrlCOch*{!Mx}LHo`NfGY$YFzaS6qKW z7lS|2zSd}`@v3>qEf7wgngw~_-Q&?;alJycs^zlFPo#0H7~2U)7MZTh5~2}))?owp zQvO!bUP z^REB4zb&bLiwvuM>^yorCUybmKug%sNg z4(3r~vNZSkZ*^c9b?1q3B{FtJ50Tdwmrcbr2C*cAi#y$1-k~`IBToC+j32d6=tY#> zXIc0Cq)qkm2lqA~cMiJTc~`wVlb1k;nj@si&8XTIe9s&8qaR=N+On}(Dj2x0CM0X_ zlbVx3mrd=eA{sH4?Pz3Kj(ZNJK{KY3vlOIxpf@>Zb{tp2_l?=Tq)pd1cL9kj>5}vx z&Z5R93P1M8%axmO!lR7Y_TxF4u%x*gy@iPdDRqh>06@=cLXmPLT(?!5=_(;OPjDzz zwRV)&oi1Y3@ht*mF$T%up`pZ1I!P(k5l0fjI?^#)GlW!LLC)@ZrS!^CY%e+8z}WBY z5_B}nFD{d@O(3PcaepD-cGb7LO*=it!Cmz2shBOF=l=%&3qj7lI?YO-bXbr5tZ<-(T4E-U4oP`8@&q10 zT^EU!9BzOReXike$dW>KzBhyJ)(n^BEXpkn`>x`s%{eF{oa-a-nS2hul1hr_ICH5r z1$gCf*(LONyt0#AVVIfHGUMgPrB{jUg0)vi$>Y7B2R&z0yQ_Ah0)?_S74~@F=PN9U z6y;;(#o2}SN7?4-LWpWBW&@EqO|mZV&Rz$VWJBMD>6<~|3PP&bxHGMx0~0^d&WD^r zYE82Ww18&`yVHvdhP0M23Vfp5{u|j- zGV0b&((J$0{(d$h2q`>0fIL7r~2)--0bUiq7C^ieO^FM-Ak9*>HOrhvcpCiSVuR zJfo{Mozi^g_7`P!@{Az9-;UW3O`#v^>aQ;)m1JDLt)Stlu)4H^~>} z8<__>Kl~JTbJ-EHmN){bdimb&OKZ=kn?XJKC0#}iZROo>Y)HgGO~>jLWqV(kynj!! z=a`|dEU)Kc6~1u&`)#`E1iR{$78`&D*y9o+`)adc5FhD@muFXNfr7OAutm@*^wj&?|id1$@!lNJ*tM7{Abyl`EL~KM`>S9GBSiTdpX5*dSvH z<3JvzE?6MBeJ!l5QycQ-_JZ_i-*nc**kTb`bv}nfl&yQ~k(M)y04hWx?c-V~mRw{? zbgYCT@+0(6TCZe9I2*9kLWxq7K7xDwW}FD4FSFJ20a20(K^|s8tcYG80%hk!GSusi zlK%~rm+efjBF6=7ze-Ec=;j2;E^@O257|T6okdT_O=U3f9x`kMDy`x>+X)1qMn(Aq zixqczHFswR`un*!?$>TD(Pbot>Jf+&OM$Pv{S)UFY)AjeajzF1kG<){H%J}f{(66- zyw3eo#pakV=2ac5$NA<{^xtUDU%#|3yFC??zO^twAOlJ9{;oAJp0=Y+&n|KuQC_Tj zjB3^PeE1u5;L__u*r6Xc6v)*c*=A*sh_cFl`a_y8z5D5P8&w8NnZ}@)(%x?0O?9nl zi8Nts(rD;uuYD8Ui-#Q3=?Skn*DJ~UwD(=1c`G|r;bw!m_6PV^dK)E^;XPyR=g=*^ z5v)z8n$7k_oAyQCth@eG^}&S=y^%;Ub6FN3UFBoavw37?nC6b$~%G9zY%NJfh*xYU3R2_u* zSdd+A5i3_A9rST6&aAfl<<{9ZmsOjMs%BYAeWllrSwcp!t0uYSd_t8Esy1Q|Yj?;H zV+nlDmZXtV{XK^&1oZig09Y%Yuy zU3K*vdR!C7q^VDGP|QvxRPhO<6bg{#-aH@gs=asDeFtkmKjdBnv`eSe%X<{UoAeQeE^vd$|(VuHGQzpryX=H_9pBpR2i{}|hP z3C67o1I~@nRQAkAF#nV+-UhgX5^{8+AbtC0aWK9<;blexAup6JEWdv9r~S?%7kSV71NY8gXy>YBrkMf(+UBUK~LBWlsno;zysg+J%eBe{W3&>=pe(xH*_QPO=ne~{IGVl9Bw(I2H zg}9qb8QErj0?kIv&hl$r!nUEjlf`0_H&%5f);rpR9!5@<6gzbfc)1PVfNJSX)NSox zwq2P}5w>Ay(;FD0$S_mC;wuAl{wt+lD?j>?IhIKtE0yOz2r-3qthD=2?!bh|1?LTP zMxch(*p;Egx~)x)T2YZNoaAtyLdllIAgrciodm)F<+mW}yYlpl=Sk zd1Y;jmgWI9ZFjujd$m00dpx*UfO_&dHArA9E!l+$hKqLZJyWBvt zZP2Z^GnXE@h&bYgsEl|ZGeAuSKlbpKbVTbJ*<>PGF4|`BWG9LEYweW}WhR(?Wb+yi zJZJyK(ynSg%O>L>5rK#vpy34$A) zv18!2wD6LB{VTdKVS0*$@0J|Yg-7e#h3fpOmTGrbCp6`^rx4(3W|w{@K5%v+51tY< z;FTs%Q9GZN_^=K_a`8$s->NfrP63Gmdn4=AS3LsdBhdpUuAK|LrMp(i?s~)Cv&p@3 znf_ZCvV|;Q$yH3 z-X3$=U-f9cn$R&T-FWgw-!1%V<*LW#sHY}$568adURcs4&C@WuLw|Y45xR>(U?IEx z3eziI{GX1jd+p6gx&Jdi@!JdkIb|yZuV)$GcB~-|R5nNyB8QJw4U?rdY(&!@dV7iF)i7Kp^3^lpRva9^c`DOWZ#cCU0@6BVZv5nZzSs!`ri<=pFZV_@?1RAkF z%JYpPD5M)EbC$zTj8a$Ob_UFR34Y&H*u1YKLw4qER$IByvxu)pnPasx!KRp5`=?vg zqz%kq1NQw|w3=WmD$!%uW^KDkR;$(Sz9eP3zN5l~WW|}keQk=sYbQWZqk0GSWO_T? z-c!$d>*O&Vr>}94Nh@sdC-&?2f}x8OME?BZS7B|>qT3MOGUmIKEDz45hytpdrNZtn z+9|*L)4QBTu;P+^`KU+b$IU)%P4y6Vh*x1)co~{bZvDX+SHq-;b4wb_W&3ezF|S^<{g zrAR!DR|3*TZ+iwsOm=DGna*N)GRWrU!jl zIdF)h!jD%;a>V)o$dod+OJMz6Lt)CGyT_~%m5pUNC&@%OG&wDLgE(luwCq=TnR3vj zX{Uk6X(TIgk^TOt7tZf!(_M^9_>u6J->ev1;+2A~yft^Lc=2P*`o}(BT$kU) z58)kUCAtGgSdj9AT~B;tJFF}AxwM^CkB3REvQ*noMZRy^QHdhbv*|2&b=)UCTZ z%hCS!w8jue<(JQmqtwv!Q|<`O6u`TBG-L>|dAe zh50*t^V9*Bcj9crx^khoN=cxf2$Lmk*6phw_-++ecO6@V}0Ek9_U&*bfG1Eo(j-CJ#p& z2Y>7L=pw>-*!9tCS&0s96OOV>0-508(+`(&x5diYfkotvN5|bh6bYjGf9J&h{UYpC zAhiKnYBp&dxPpV2{0K3)&Dt$?z-pun3Kq+W1IGI7Wk~i3$CoX_X;0VS5MrLF*Yf!K zkQFSGMyUF_lCG1^vhpY6hr-oB`oLLs4cw4rG3GGeG4!x)$$P+HAvx3o-FxEjCy<)@W;d2Pp_L`mg{8t*Q!Ai zT&-CO^4oDu4*hFkq2tscKq(4X=A&A8Vd(Fn`PYl@X9D$()lDlYx!WN`1yKhmOe zHvPK#0utG^{<~QPWy8FV7^lWlwisO3fiq!eclmbNJH^Wh9C^#WV3j1?DcXDgGjyN4 z4H^5qnkNg<;T5W!ir(H!DtUCZVuffjd;I2V4;qZ>NLH0v0(`RG`YqdR zIe7=4-OOYztA8`$SRPy6$c)Dx!)c1hx~1eBzktGj7)G(cVun3Ep^E&uaQntQvVi#w zj9p2lbvjF+z0QF095a4Z`7Ti7O3aj5tVn0ueZg4zKDX6!kJDlK}D z$Y9Do#_o5W)wzkjlo$6yAve{s1NfGCa6C@4T@YzkNF5>GM_=VoN7|@la3A};rCX@5 z7oxCt89vI4*(WVQXHQI93;2W3lm!l%O^+UvT7Sj9`8*t6-+GptY7DwCv)=kCkpKU4hrg0F*KY-0 zU@{g`?=RS;-YhvB_{Q)O?}Mf4wXR?)dT}<$)`Zktei@rX#hPTbFlD#Frst2MM+ZzD z%}|+~>_F8_zpkvveA)IwpR(F4a3+GC*BeiM_T5}042yYsHdKvbx#bxW<3r|8D_1x6d0V;>u3;+I)-`pHesM~vL zErCNgEfy1+T#h`chE&rFhf3S&z zS&{YO*~f|6t{5unFu6cMOULIo&i>l?^SfVuUxWz+rpWoue4d3t$$bUJ<-ytAaPvCE zbi8pE2!b~VU}Ag2DoA6SwFzck6R(2d#ZeT2#e9G<{2)rm{JkOMQ=eoO=T8QBhewY_ z4q(ilJb0}%SQ7y^>22eKClyxCeEb6Y^iSqGe%iVqfTE&w?AhO& zewZsIh~m!D3pwdM+M0547+Dp`5yMwY&l2t90VOD9Av+X(Jz6Q^wKJd1K3mv&oc_JR z5kx?W0oHp^{a;P)cjs)u6cOGKa30RJNCp^Lh9}m)TszS~Vhccuj$P;`5~~jL-!R+( zN1y&++4HutHv<&$R5}#pUBmbPlU@AL&A=KgmdC%&1BRp)b6@11G#LBdvzoHqi%DEC zzg5wpCo!3TI9?&n_i`yIEgvv|1D5<)ZsOqFIsiVoqqee_6hFCB#*{kn-k|^8-haCY z!&4U5E&)y+uX)4hCcv0qrM|O@Jh?lzf!!xdG^5Kf4u2rMb-ek$ba@2iXECY=3~*wX z>{pD@2ieZPp3H-PI$`M|5cf0VNMctQD?DxLh6#jp>)z%_4n^|jVK5-`e4 z@Y$t6~8O#Hna=HG_pdhLno z^&T}F@L%f1WJhpPuIHheK%4_0oF>Ojpz3c0$_%6@Uvu_cOSFqV02O^8%YoQkL=G;^ z2p^(UKXaNw;`qR$zvnP#Kh|GOFp1{#J$CZ^y9nT~+<>{vswKJg+*=#2>%J)6GUAF( z_84(Stsi7aLfCw`N1^*9n}yHWU1H7AbIYr88lrc6TvR9;7o6 zAS07;z(IR$ZEe-S*>FoAus@&|OoESTBo9yoq8H%Dk5wK7=|a;NYZo*R7*{LvwCLZr z78hNQT&hh>8!p$nZk%G3Pggzj(Kd@g&)_nv|9Usi3!J0K&gK%V3oOxUm>=-}3_=-3 zkkxa$Ue7XgZ6v%3Hj!@FHksU$!PpbUIc$>t@i=OyDtiNNceD`(O@gn<-R6 zBCIn>IQD%>RuIsMn(FJ3LVz=@w=g?C38~o_=2I7LK6kd}o*bB`W=)p=SI|6T0Jw7R zDT$GHo2K@q{XLeEa`7DJBk1D_6iD-$y>);utxqSIXPMRbOrMsxh0|i?JQ2{2YnE;T z!){8O)o!2evzxr+$0Y|0=z!pBf+;wZQ(HYj2Bcd}8W(n>htb$BN)V7vzddj#`BT8d zFR`Z*59U$hMC$F3fe7cxH(XhVX(c~x{78;%5a4aIyz@YzDsca$_3+bM)!c)BAC7WP z-1)>r@UXm7zX9S;Ryh`eENcf)?Ez9DYf-1Mv4VX~>FpHt?L@hYrkaTJf-2j4fCR+= zR0>-MoFR+J%}b)t;7aMI^i;geeC9K4K(-8>Xt|$b zB7g$Nh+#oh)Q~!W>axKy*9`&0_gnJ-uew&YOH_?or-%1c&ZZ=G0rKG1r=^k!NH@Q_ zNb8kt1FdkDN49)nDjQP@z`>j=D6zcL>=;(JHw~8eJrnKA?t$!$-bG0#i!qN5VbqlW zY=@>?+65BOR|f_T z(n`HYm`vXB>EYfBPhRA_E>G;!);5{lz&OHgkLOf{2W$Zv)4B^152#8p$0T!vbcDN) zr8*|(G%fO!_>W6q?aF)GNQHQVVetH#o{GsVm_$j3H@whk!ydrI3ka)#cVrw4_qKne z?Hl52-}SjDv}Nt~O#5!LbTwLj*X|(Ttl_AmgsSG!{1k4@{M7D`D{V(##mL5ebSI``Gd&f1`iy+@og08Jr1m%sYACK*;jYZ!{2TS z$g@wsMjvEKlLk*k2%Wp5Wasw`=c=u6kfc*Myy6{+9>~or{J0seC|xL&8-XhoKFHRc z+eM3G>}){#3k@5@IM&I9;ET~0ZS$ixeSvRuiF5bku>eQayOxw#kdP)OHUMnn{Zh0t zneFTKK}Po+4CB&-ee5UyxQS(@61lUGw`OGn@IafZU}@jO?d`3R7AwJ8#DfSg_Q{$s zw06>MlV0G*q9&_3bs~kXtp6Ok0f43Fq>*;zmPhc-e+wMH@Y4HBM?z2Zm!%0f#gaX? zc#P=F8B(8w5TkGXMvkP(s{(}w5ewZzy*;!xGan%*)X z_?0(rcZjs}+bew<*Z^5kP!}IsR*-|R-yG)fvPS~PJskR=oX{n+)G@37a=>1Rd9I9O zs}ly}2(ab5Ra3?hW(WQi!r@H83V&td!=_$J+Mj;LHi^0sWMKPmjmM+ z+!%VESMYrITwJHUQhi%|2gFZN4Fe>NR@9(k3fr^yD-~%%6RD zIvna$DemZPRZu$6h7vH6N=O_IRUIET0m!r~<&i`hNY(A7gPyU43!onXUt`Btd-wa| zbhA<8P}N9wwcLD0%y6g>)0E}8KsSQeiqvcW(CbKhwA|*;>nK3Cd-SM!?TN6@961E1 zQ6;A_JvMlfgag3Ztz{gNg`I|e&S~Q4CCDbSYtS@A&YMl!=H z({gJHsH2z{wf#P%7Er&kUDKHJ%s*t}Cw=nof{jJPIRk*MZ5e_scTebp473S@Nr43m zX7=2gN~W{UQ{|^NQfVJWa%0ntBywTtA|=$iuCJc!s5=7Ntc;cfD|+<0@3rE58;)~j zU>uMFH7hMwMNd*uP+Y6Ocju-Kb7iP|8&7tJZr`+XJH~p##Jt&9esHd=>Zp^>H$BXi zgk5uhJ1ftKvZ16KpSCZ6bqZn!^|xAMAHPG%5qFx!d=EPNevKpvl_#&+iBQBnKS#&3)b1d0zkX z|2wZ61l_q6$SCFYbj<<47z`XH@VS1(q8dXWoM{t*7k-{+`g{Q1=T3(d$n7t*ICw9X z`H4r;y8MFHO6u_?it)9BPxco|wqe+@+CM$uW$vvIZp_H;Dj{d|@D$jXVSvbu8v3!!?YXXv}qPs4;$Y<;9WW~?(mv%q0zr^BmbrVg-`gtOH2U~BRqNK1t)Oihwi&FYfHSXX7b)d<D zHs&NBEh(E7&(7Hto>Dle&bHzE`)P{p!JEGxG;xTGYmSh3FY;FH74OPNXx%?p*TKUS z!gjVrykM4RO3S}npKfM#P1yl#3j6tF%JmN+T7HSq*q-Ks>+p8Q(^3J`l!((6i!W=$J5Yu zb9%;V^h)%HZ*6tm#33h*aG0&MjrYB!Dv%x4J`X$^Q67#Flq)U*!PuymCdW4VIPSvs z*&a-+Q1+l-NQ2c<1Xt6BR7MRP?41O@r;OAa5pL^vQdClgiV*A}U)0JnaCP9PW=c`5 zw(QSmo`lY8jDQ#MuX|#QK1N(k(wNpZux`p%GOuz8w^}PRx{{S*j+Qw-))(n!pmK)1 ziH0>r#X8B?_oq2?pFC-fVG3vxMytzNfqIZtn{U2L#uI+{UO#de@tD3UPVIe3uRrk@ zJea2j_?2YFIiTP}GFo^gdm6K5!zJ1kPiBwj3V^r`OKr7jdqmqQI?jS49`wXudcboN zkJj;4f!!D>dkE=LRkR46$xdE4&twXDKV+gtOW(NB7fd_4slWFNT{lt6YG56_P}b2V z$5;c=5w-y5^nv6>8C0eqINp}=XOTNEB&X?#eq>La<{B3uCj>LVg1(-F5!>D_e&fdt z)e}u&L3oXe3r1%&3C_oeKM9H|M&HFv?Go){!|lp*?5V`c8fz0}VX=;9eXq)TlaWKE zEVrCZw25b079YiQ)=-skRf2zMi1d+TaXu0kb9d>)M=Fso2;0m}f5Fm==(_7!C4L4} zlE2ghm>NUk4qV8+5W&&{lW{9n+p?Q5TFA_0)!Jm5yO{a~KrBbx%R9tY91l(+zwdI? zG`x^J8|W1ys40BE_D zPbmKa`Y;|s#^;q&mCl zh}4SYs#9$q-C<&Ch(wDNAg8cb^59muV72FdQ8xhc3i#Vn(YP5KSW+#1R2bpQA7b^9 z!z48!h=IOdQ4O^L9*8S&Br2u?l)O9tJ*!EZ>=th!yhT1I+q}Qq3v^Qes83V8 zpCMZVffL-*Xs!_KCFi_05S_R;&1l@zi_-78Ad+YpiY;!F`O8Zk@@yKYro)7T}DDR3Nd5^jo zz|(A-J&muN&SCqczYGQPuTSqN`d_i``_n8?q5bzuCViU648FgunZZA4+V??2tCFGro(njy^F*+Rin{25q+~^(iGw-SGCankmlhP4 zPRIGc)Jt_fK3iV~L!#fX!G_&lG7IXEz%EGrBh^0HULAlegRw>UM?r3liOoAz#Pf-+ z!KaN42^tTyH3P_pql!jN1u@JwusU*8AErV3g2ptb(Ke2^pY7Wdp0~VR%m@HM4i)Bt7P+B;(ar|OW2o&Xv&GOq}#cvO?3_LO?!Zj zVwbf|&#`?LAh*ZCN>C*9CuqO9(bAfeTnk`(xs=P4Eh=>d?yEse>!#X5n@sP*{7km(X42`e1SB zm^j4&7`AJT!h~I6PoMU>4N32LuADaSdY5!c;;;Rm=TJ#))341P1#_fToujk7UF__l z^lsLxk<%=1YZrEKBgd%s&X`9j%je7VVDyJODqiuSNX-)NtdZ$d^=!+`&R(WAwyMMF z9lm%r$Z$u4hqFnvT(EPvL!y#QrGi(`=*-9*Xak;VxCBNq09qiBZdsrzy<7H@7hN`y zmF0Fge{lw;QhPo#eDCEFnx&&X>W*Dy8LezAZ}BdAo@G{UVv9IC_-bh(C683L@T@oAMC>R z&PSRD#KEj;Z6Z(_CG1TfSa^kD9Sy6veb|a&*e_kiWXoNy5oNF34b5iC+-%;dx1y5r zc@>##nD^Y5PdAb^OnM#Z^?EIh7bc%Ne^5{Q4R!^>&2L;!+Bf5`%|l*dWXQHj&b5PV zlrYogy=r`>%_-!7YgV4v=iOHET|(^G9c@ya8CY3Ew7SL51tWhtRvMJ@Hwg3Bo6Vok z+{e*Ysyed=^Mb1t9LHm3MN7Ce)t)(}EZ}j_X=-1*i?=+VgbG=w6hadc;ycZ^e*_C5&;HU9wojg}|xvs@49i8NA ze@4c;&pLAx1Ff0T0KU97fnQh(9B|r7ABEp~?iP)PGq4??+r}_{V71_GqfRo@n2oTj zx4ER|n~#3WKBQMsL%Ru7SeuHC{FY19g@Vq@YiOFE9Uo6Y219dbNppXka2*LGGqZX;AUSXpTZ;>!U2OQ_sm(j(lIc2oP$n;fhxj}HwPoM|oc+NT+ zc>WSScEa#VFgOQ-5!IP&7~6B8&A3bT1$Vs**6`15jZ`XbcT<@V);fFK+*#hV_^-Pi!wt zhn&&j<3%ex#=mL`8o|dVZnk|?O-{=LtmBofwpFfmf2o26fkR>YvsMFLBaE{B|JRU;f5@^({4SM)w% z8g79aJB$iAb8y8Z;967jcS8-q)s?#%o_;ngQmV=6TDZEe@Q2P?GCp=B7^M-wo@@e5 zV3e6RmQ+o-d%N0CD<6?*pG^0zfDhG(ALp=#dlUTcwI;JZzlK@30f2)XiPyVANzcKr z+pLyry4ZWBGoGulVHg+HGP(#RGw|iQ1Dl{8DG1Qewo{SA%>-0E;tiK}ZjL<%(bFX? zEn1d%1$=4825B4zKy&`f9SINQzgzVHSaAEUL_DIU(52**=pkO91=q@m>?*mfA;BrT z=t8Jd3s}+_X4?+rfvvuhA5Y)-AIkQRe`?lW%5M+VtX3_92qcLiw6i5%#~1g_>Hza7 z6?A9q{Jv6*Z5;@VQzWr>}us6wEqx%Dl56Xb2DsgSs|f4S>NL ze)b8qG)<^f`oq6;<~2BV5~NvI0Uo2FI|Pik$_U@T5h*xsX4yMg>K7aqx0PhF)VU&=;-)L*~ zJJ7CAB@Knf9{?Y_ftu`ZMlTEnl^p}kOixDg6}tiFbp{PF>@xEc99hhEYtx=fUbjx| zZ~^WRgfFs=>p;6(0JPKtz=?b|-p>9Ac)^ew1;I9`OhFTMF|7Ls*;jfT9t`fwYd{XK z>%VpXRXiFbKAXx1?W7OSru>nTSFaAP5G7BM9A7%x3 z%lj&TN!#moY?Jwwp&&Xm4o0ZnK>s_LQ~2+%M$*_S=ePRl5!vw$eVL|kdHM6>tor|W zM!8G3Km&(U#kp^r2mg=b>bilL4<{e^N2vd=&;GR?WWVJh`}h1=916k+Cc?zDb&29j|4Pu`Dva&W5kq zbGgR+CVMWI{EB6f@ik1AMaDN_^8Z<6ETVo%F+ZSQRF^9U{*K^ISY}$dU;Zzh CCJO-o literal 151681 zcmeEuXIN8N8!ci~R8+u1FT$W89i;aG1VKPR=}l0PULrL>L`P8RO7E!jCcP67Dbh>m zkkCU9J=ElGX2vq#^ZDI>_s4za@ee|B&e`vN_q*4-*4pRwT{XpXXK2n45fPoctt78W zM0DyI5fMa$xR?mF2&wzPRfM8pe;K`HQUr9;f*AEB;ST}(Hr#vo* zeqbsRa*9ZiU=b@5ByB`QBtvvtUiO~bv8Az7QLOi)dB=O*i7!$+bv*y=>(^J1fdI$L z3Wg4R2icZ^QJ0Ug%o?2_M+#o8lOb~<4+(e}61PFhduPrQ3$L-VgIT1)z`sh&_A*XF zy*y5b?2EBde7%_x|j7jDh(SQFT0Or%^h(1pL>redoWuM<=hz4K$ z@+yCQMvR<{DEA$02JQd6BN-XdHtGL(BJigqH3;O{;`FVBlmGppzkM6>O!D~uzE}Fi zW5>qD%0nw;e-(hgu64|`^A`d7cCQ6_8KN>ZyN`C?e(~R<3c@7tZ8gB=oVT>GPRBnJTC>YMS}a|3IcwAUtQ21RfX*C4mI8`h)mh)H^_rmUjC0RCP@gg zje=p3C{)cIb8f`JdQlStjD(nn#r*hD%YghKBPSZ^u@ogm{^z~!Uly55deO6sNkn9( zIZFITS8?|^@mWglJ6dUKIX)3ji2wX6mC-SCTXj(=a+aM0NGcqzPqnCxqsB&e>k>SezjLIoNO%A&G^J{m8Dc$cX5SWM^$Fo7i_WX*O{*1_!Vnhw8rTGNvd&xV;Fi(HKwS|g_a~+DG4bm z?A%!?wj}}eT8M{3Zxl<%Twu?e`*aMaC{t#+CzRnn+mYeE&7I*s+<&~kFchHbtQ$;3 zRhiF&uK#nOKLWd;oV1@|t%W&6+>ul#czxNszv6vutF5v?w`WXL!)^U!hHbr^=Hkgb zwR3@7EH3odLCBsFeX{tEApexpAtzd9G_43t16l(gFaZ^c8HW={Z;Y!Jh6S!{lw*Q!*1(PHsUe^k-yo0<6O+!Wq_^`-~ zh!Y}^8=G4h*WnSvj9*Do!k<5R5+5FuYWpt6VmU6xVkNGn+Fz?pbkrMjL31iBuy>j( z$S`9=>=g)-%2^U39w8VwW9ovi%h&8N9)>RvF%6t{c|#E-L5p(p_++;QYvm1-Qjs&*#~ta!R3?7?v}>y7d5XwNOqvr$Yqu-Bl_s|)^(Z4q@5>OmrE4zyS;PM8C<|mB z5;eUCF;BX3g2p;uV_5!|iTBsU)>DQYlkgN!{Pi?x$JVwBqvII7cP1X?qDM#LN>Cn) zMcub(xeD!X3dz&H=hB$0h_x*la^mUj?S02xwfgyeM_aT&Zvt$kd~`3%8dKOQ6?;^Y zC)XGA5Z=R}?rZC48AjGBpC%#7yA=E4pAq^w+p`qMW6=kX_vY2%Vsi~?>WK*`{B~*D zRFF7#!`_qdjZSIl>isb+w;KFlttlg zqkP5W8xBqLsb&m>!DB9YK1A+3yO@Y&&&TfNbg5FK02G%V)V1Ev3Lz zQV|F35S=g1^Riu7Sy9t5@p2ZM|M;#aRFNTCOiZjt&m#W*y)iR?t{yOs2=21+=a;r{ zBld`*{guiz#No1i10Ewy*k$i9Vkhi0N%x!+9=+=sSPl~&d)m9%C3hhuE4LzZu%uT_ zJOv|(bq`-)_l4nGP4JqKMDrT-@$(`~w&gPshx?M=W%Rwbh;7wc zV}tvtY~5nMg3NmMJMTNegW3-}HqmEUZRV7emZnJ`Ok7a7i=R(RRI|=4fBfK}MW=?p zxTuIz3w@H9g_Vg&Li25h+^-{d%!eeXk*+o^&PlmD<}6RuClVq>vh^H$FYEkeFhX^{ z;MEJ>t1l*D8TXX)?`<rHy`Cu)}c+);8$sW?e=(X96w{=8afbY~FqO)|ElV~^S;DmiN z4}W$R&TI7r;~n%)qwtLbBs;s77F-Q4$71a=?}XpQq$#hu&UVEh9n@;qla4!1H#+&8-w)39Dxn8GTqfydj?u_jPx zI6a-VZiaBsNm!Czk;YF+zZthDtv=k!gGb zC}jt!j4Sv@*TASzn;#etpA2FGk~0Vg80oa3LW?v2G_$EPMSV+yn4k{@J7WYn;vGJp zZlG&+T<`Rp6`1!oIWTjK&&|z!I_`5S6@lN%V=s8~;T(@gxBr`QT$TIA8!@amuG0TO z!IX-+x_an+Pn0|6^8O9Dq5Dkup)+#`Z1VN>E0xSLRp3q^L8_IXO8GaG#zqcC9xuPM$g>0Y%S9$3w3;OZ^f<)>>Uj?=TEx zn(krc*XoqmXxscuS;MQEC?%;HFXlXdgJbLPtqS+N^a=~(t`dAzSjZbhVHqtP7hYPu zY^$&;9Ha-zT4aG%ZaeLY6U=1>`W1UG?`yY@hp14mBVPKQrz^^*wGGeR6s#MfHx@OuR@<_4selDZkB?1#OH_?r#9qmGPb1`qX{C|LPJ%$w&ZtHw{f zh;MWu&iW1DQ)SQd*M6Cf$OM^(vD(Uy7)D!gJBHRd#hD21Le$Vu`InMrJ9_14KJ#1f z;;6{diZ2dr_2=-7YoXx=Mkib10aS)bUfK=K2)hY;^sppWUiH z+#ZQ+-m6|JXx+;z1^=O3?C#*mEqkvU>{7FEKW`yV%Lq}x*I~lW2ib(vjg)a~tU zJn3|^T9Er|!e*NE<$c#t$EMBjJkJzev$BPOn(bot*^Ax_8EHNDb&T$}sgJ^$IUu)` zOotF-EKkkSuKsdj$f%5b0C*9vo}$OHMF>|-oMR`B?K_5!k`{g*BF9c^^GOz7Owx~E zaYb+xSechi3Ur8kfTP#7k5CF_y@X;s4}7=vP`01yWVweOY9`+N(RTGlJa$QXDg8sx z4dS}|iHgNsj$!gezRtBp6T+UZDk!hIYt-;BaR^tGyTzJhfeT_Yg=Y(!mHow`Bi#Kh zeEnVGCFp5IA#D8x={WqB2|;kVWW>%Av~nB;udl1dgawIAzM4$i5{%h_u^`B)eEYvM!`#i z%})c}ohUhVLr|Nk!dnBuyD*k|7X{YYUQn&k28O}~jyTYR7tO|4U!l&^O1*PCCnty9 z_c_Bng{{}vqH$ygs4Y*s6(}#u8FnT}R!Wo64T!IlP3;~31h5fYY-jvMZ#gC*b_E2b z2(%K<{5oLWcEDvS#e50i5S*~~>&O|v%e_H1A}v3dId7FNKY%zYws2YcTmJw^{D z!{#?JG*s1xZqYtMd*M`%w+oKK`8gokvH+~z7#9uu)u5T~au9QIaapFtyC`El1}fhy zje-4IRH4gRWQG#4(?C&wz3HIxXs7B0HBNfE_X`;^aeoxkvpL`EZ_b=fGn7YZ=pdWf zRwv#iQ6pDV%XFE=eAn7!U!S}-F!|F|g$6Sbtq-cXF^AhC-dS60y_sbnLkQf{5T0Gi z30rWv^>~XBzv_o?8v~nS^;W;i^#nIFvDql|8-6KAcWuB$S?>ZseH*dYm-pssB8#{P ze+h@2JdV*6YN-IP?PiS<9Xc7v~IjU`w@^C9Gd z@GHY|dD2iixf_aI0{kzU3M2qNRM=-Jbe(OBZmoXm>gpQ0B>^_4q6uGG?ozu)7Ht!> z(Y20kfOv>uzAJV2H5BcN_L_fvJ%Nqi_k-P*Ve4^vb6TJKm#KCv4f_hkewms~GNz+v zzIfL5L5cJ3DALf<`V+0U&{bwO$uqg8*NryEfsHfX&-p8ru}5C@w*Ca4<4RPd=#`hX zx~Hj#J+eO<`TcVK{|wFtP5>HPYSq+#2^x=@CZBu?G9m^J%Ypw*+4rQ%>SwP zf0R#v#7wgxplMegcK!iR{u@5yZ6OAYCF#=U&r9{_X}+}|wL4@_p6E(Gc-QAVw&ti) z^p?nH;!Jje@Q{N+hSawYegE{gA56ne#0-oIr1++4)U*_&+3Q&&9tjWs4h9(x*f`_c zB7VQ@HMdy1B)np8IydE|`u2;O_vX{t*o|y{s;Yw8NbZpM%%1L6iT{th`|0^f zWs@GYkM(X=tE(6V9{}!PJ?)EW9d-#?qK(Z&A)WcetNII{z`^df-N2I2n=o(_{W)I%>aAydoV-J!c#~yiSo~xS%?d zTb_ZWZmkMC;tV>%l*ioh<8&^THtoG6r85scTFmqbYE6ACUdMgCcvjW^G?{J=n8>Vf zbiT6a9S&UQf^}PQ;2n|1X#P zDO!T;Ohkt7N)tLsfl|IR)JQzu9)Tx=jkoo;lvSC z_-@Ny9KPC0PAfMu@?eN;F+V4R3x201b}s^X)BWq#tYt?2BM&RK z2En~hzrUp)wFML^4LiX>eTMo>Zmv~>Owe9 z-aGjc_W7hAQ$yuMIZp0|j^I+cYdsYEM~$nTxAaXRjH|3S<1|jrYGpqFCr=Q6?;bc= z-G%dhg_HFb-b7qqubK4Ha;DDud6iM6fu%aN*=Q;Bee6En6l~5NDk|?mx64@Ypl2Y;OhhUEqFh}us-Ci!N8qBTj-mVWZNA4mf6Vk zmZpvME1AOIpVd+juT2GM?z1%PCmoFLKbUuwhd3)DzFfFvrnfK_!c*0Q^+krD&k1bo zGZ7BqD~Dv*Kk%+!K00eoeLV1PmVJCQao-=gn0p*yXmxnrlJWa5eKg3}oTb7&DA9?4 z+wV9QUz1X*3;)>sh4!UNeg2zGl~a42?lu3|%%suwLLB+j_dtuA=YI#588pCOcm2 zkCR2i9vos&B2N`1kXRXp~FSvh7HgKnyh{U(^3-yn@9-yQ236oY;BsQC*{oo4PbxRJJ>XlMw5 zUssCM$&6JEjJJGOd@YUL_IE*!6LU<&U`4EIQ-?L4|1ld=DoJsL2L3V~g{3K|s!fUg z;Cn66o>}~C3p*54FZyS9RbUvjA!nbCbzGdlu{3EOK~O)a4T`#L_5$&GKDa=Hj-96+r7ni2W zh=k=b9_7R*Lm$6HE6M|-+`H(G8X0%ku*ad`L*ff{DnLB*X6V|;szS^De7OZC&U$xfAw1a!^z^tgR*k*(%N8~sKB)YwJooa#HP z_c=5ggro97*qK5B`W~&-DX=>h66xg1Fsw@r^`X+Put3c;^ti71){st6i+`NeJNk2D z1hqMeoANJ!@PWN+OiQ_w+xRhOe0SA!nUpKUCs7X>-yhd6mb;(41~j~gXG9LRV;6FM z;4Uyklmw4OXBCGQt}4svE~arWA!p<{LL__IK}u@e?f^dwhpd`4 z2GUW4AnI5D3+~h+CRKHuz02-v8SpJ*y=-z1fBkbe8mfu;4vO z&?LQ@iHwvHwXk-PTyHYFt*WK9sK1{Q;cSAUB(-~znb)z<{_4yZ!w6}XU{SUI8gWG>fUYY}@!XSktnW_Us|VB8lMVBLUt-toH-;^5KBufQ<$b@oRM?r?%7gH%JzUJKwDUl1 zBt#Am51$-0$)jcTT&;TwIGZbD{&@R>uh#_o`oytS0%OolNrhM!&;k^7h%F-T?OUbr zUBudZ=QLZN`cV&4AjO; ztF^~_0Z*kfBxonaBICVUcZzb^Yb(nnzC&t%Np{2=aGRS0^y!*_pBx5yMi1zmqK?#V zpG@RfiS*&T`g}Fk%f35r4%e;7Xr$#OF%zcM13F=YqWR5Ub#*^FCneS5JZ%~lB0~<> z-^IT}ziZhtodUvX9e!#&zD}y{%@&U)&&WS^z#I~^INJ+dZrjpU=SNW%B_}_ae1B&= zp(zd>f8nrZ=Ydj-tmz}-vzso{nDIC>%eV?XZ{ez|#ko+Kjx%=gwQLn?f8p5x-&z75 zOGpidA51l5uyjc<9Is`W?-?11&8^v6C_;PhryOy}^H9)AD~%#Z9pmkjjvOCY%**I8Ja*0Jwx+_o=C zzLynTf02!=Lb#YPwU{xrLg?@r7fvZTXA-ICZ*1;%UE}ZdnSR!1plVidUe(^CH_0oS z|4~cQ{*taXQx$#@fzKKwmy<&S%5T0+pz}_!#JbXraOL^4COf~29k~ZUx)v|hl>|iB zT1ag~Z{`PkWtX9l8Cp$C6;sQ<1L&|EWXba_^=iu~=e!qUkd6cJG*GvL_G{*9=gx)= zGb%5a(kF^x5A|Y-b}O}7#dy^DpDUuhoDGv*yS*|b5wvE5CNFE>Kw8E+Q-fe?%JCNt zZWP-dlrVFYC0m-ctCupf?EfYFSD!sn^yh(CV`L|6u7s(?)>Nh!&Zowv0(GKLmnX7y z9)}=!EzHle*EFp*@=>k;k*5n@1C*T`stFRJ4H8*EnNcNdlLR!#@&fEH5l5^EF{rjMv|8zNf5a6BH8C)}i0_u(n?0Paxm) zTDCTc<*r;Rxb&Fs=DPEMQR$a*^NtC`!3@tlAVOf>sU&2Iu*dB&eJ0o=-o2tK2IkGp1Pl9@wn`0@sa;>grcCij|+ zv#Tp54wJ&;W%;IU-JiQ6ef2dSvDZLjQK}0HHpX!X~fRU>zkLSeMQ*liq3FS{LNRQV%>3)9+D%DgqX(^)0`Buh;zKfuy63z>+J{ zX+Xj_WZPOC`Mzu(33RO9tNTFCVe!XX^6E+*2B`qy82j3b;5=oh6pu9yFgv(HY}GXq`uTIZX}5JzE@+Z0|)aTXfZtQ|p&1(1%B9O$$hseO}s z-2)`KO#k$OI@@m_uH z&Mh>{i-RyNwcAJaaQ`@({8JF4XaJ7+xLXxkcKLhTo^I)*iK!}LBfRj$9>iKz#M20) zAT6S%ZjTdKc7|^{@AG_26%?Yir;l1)=EAwmkRPjwuV9e0_-sH#rRS57maBX=*V8qo zVC=ePYn<1u?dgtNe{JZV0Ag!mZ$2G3jkw5XLhz=(| z2&9nBYaXrM>{0Q@687>4`jE*101Qi`iLQbL5&(e-?TE5hw@d zjy90}8$eeUvMX}G*>a24_FfTJ2YvlLMIwoU`z&|q_4@e?U7{3EW2ltNs;jF>qI5>z z#P0?_fg)eaxP%?aN+Ko@y2B9bD>J}XWT$(;LIU*d>hCB=@Kf&$R+sXn@@}m?2%hORS*qgZy~aI^CD9k?xu@^=^!tHnSWEvs6 z6Mz`Fb2it@B&K4FFXK^;SCg&uHT`;(c=Z|A`s-Q`p+GB>OByX?k~UB5*zhG7#1zf1 zV@}WoE{b3O>(abJ#Gzh#)NkD8Id1V5Ycj81(F+n^`Xoo5o<(}@0LVvqR4XN#_U?&% zO}3DY_ww3~4@b3?ACf4ViWa!ByxW-j5S+VoLy9G27mq{0U)%PXm@!N#!&r1FH_qu4 z*f0Wf5A!qJAr_7xgpi0eIUx8j%dUw1G5m7)Zos~9z_>E&MIkk6*`*z0zE|#6ELj!& z15Qi+Z9NV_Z9OT+Ysu+yA@C_rzBGHAvr{69kUAjl z72$ae77%Hc9a`35a;S)+gM_l%Eh3TU*&36(84MMoHBG!NO?1#Z`G|EzNZ1E?Ux&ED z9^BxSV++kyfv_vGgHe=an#_Y5V7&%BKWbo3GkM~F3L1%0C zN+Jg?G`;P2R)ngQSh908>I!Q0HK5&^Y#DEQ#*u8gd{hwjDrw%fHo`q~?3sH#(x%yT z(Q?8W7(2HWuW>(52#%I%lDwYcggY>^8|SM`Kl?YsZ+?wz=v}C@Xcz@`cIa)r(=`pe z4dJ5ZeomfStCdG0{z=K-3cMn@`3TtcmLtvl%SzOTG76#~n>MyJ>W#8S-gXyRc7P#+aIp56)PhcSxaCtn(6G>vr$1}QkWr_TQ{{2@9UE=gCdpJJ5>bAL*4|@8 zb3pl;_Ie&MVSOYxj*UA6EMb*b1@|aq#}+idKFmdFmlMbz5+dQdkqbsD+0uJ+AC9~@ zWb~c-DPO8{!dj{io22m?L-!8Kk_alsuLH)L+l(}hB3!}ehc#ubV)hpy#_&x$Ka_8} zaN6uOg>7KDkGc24%!#;D;&3ioll9zw&DNqHbXu**y+UndCZWpPYN@SOWr0VbzeKC^L_U|pTQJ0B0 zluzA0MSgX81Hc0c>Y{M3<6hdc8G&AWa4SgIsT-l^DxPML@>Oqcx8PEahxEyj=q$*e zbbK^8Pcp7>mzUq3rT!-WE5$9#qT(E1 zcB|O6O@K}>`*=9O_72hQ9P0tVwsX9M|B~C*?UPm}DUy5y`?eGGiQof5gQlrD3=Lc&z=Glm#fazB5 zN}CfpIm=_jwZ&fWqM+{}HSTm6w9)>dc`B-AH>^=_L(o;?Y#5YmCxR!%5*8OmbN!su z&3?C)NjH1gzHJjwWonY!8(}6zaL?0EM`2D@G5NMu(PP4$qdQjB{S-`p((o)-SA8yc zGD)s0q1kQ%TQ*mLjJfWN4!~mUzBBaco4^5JA4J-x{5rc^`I@7Re|D?#$;S}6NbVOW z&PL`658_D9BzP=jG^{iy1O|*KcdY^o9@Ww(@*#%kGMqhPS?L@QoKfBVo#DmY;@~OLk4UmWhyyiST~J$nXuz=BQk={`;%H_roo?} z()2S)oY(e{f{@+A>GMUSpf!7`(AG*Thr0FpPEWH(z@4uonGK$Kq4b9A`PG_781Ra0 zG0AH6igTWdz~eSH7X`AFBj?9+TfS*5aEx>?E0u!iQu@ratg&pBJ4}2AdpyS9^a*R$ zP2wal4eLrq*SL(U%Ft^7L&1ja5r=Ok0yu)m#}C^qq0fUgpMZDCV zdIYek9*Yo1s&o=)PkmYKFxl&gd}Gr~rMU~t6&`%q&P)Eak4PuzHs*lqASgd0s&gBl zLfh^d66IU7E=$21XkeC8h@Zea9>GXId01AWPm~I-s8`Kk_G$QD{uKDuBEoKS<_KDL zM?}HOD56YeA+1AX(DfBy6j2ClE|CJJ%QrGmi0x5(N!fXCG!}RxO}h&3tti|R0ZoFx zsayX*{43H#OUP_A6fW;Gd;EBoUvA( z5t8RmQkr)H!d!O$YZZO^2A_@7!>mTbIt{ZyGzqG)>nr2ooZ^wOA@RuNYX&EU(Xwve zZ;7@@h&J0D9;neqN>BW8@r})3^L7AdS94H+y8}xI9cxGVS|hE_dB*(CqxK7c-$h_G ziE;tPi%WXV1I94hBR|o8-bJ6u`|P2X2cYzM)@sRztP3M{RmtOW?!7Wy1uc96v&=!D zy(f#;q5)bt)W|5BR>YSUk4%%gpMfG!+mZ9)7jzK%q$!5fzM=&$S=^{uiPxm}x`TUE z!1OZj9Qi2An|Xt-zTtpkZq07roo!l1IP|sbqi{85N~_KHf+m7l#;NN%clc_v^$Im% zFxdU&dk_w?Qkv!!?9#xSMR}Q>80%d2l_AzhLru~} zufF>_Thi*}s|+?&^9zgp>lLKq`ZeOie}id2)aF7e63f_9G;w8C;CMyaF(o>VZ|?#) zIu$Tr(bQ;}=Rr9N4rZ*-r{xSRc~H!)+oh+CWbpj8CY6X48QdQd4vIrX^-r&QPnj7Q#S3U_R z2p5ZkTbJyq6u^P7*#aiE&+pP=o_o9X5kaYo%!0tpm1dhWgKR)y4p^XC5i3zJ6)Z3X z@G82Qd{lZdyKrycSoCpv;#&PPs*|G6o5`yJOB7dyiF1IvQGHKciym?aOr{Y4*J3{k z4j2{=nw-wH1H2Tw?_`j7j?HW67qYhS?RyD1JKdlCq&AZ;QFrwX%^x?qo-kqWy%Edq z%}bWjQ$uAYRxo4!WRA@-;m9zq5F+e593kU{# zUHrR~V&NnsD01ODYkOuu%S-#zE-YnnPwg470O&qlbj;Y#-?*lowpDWNL`xF50&u!=4>c1JDw9xVccCpVt8wWIYifp`7WJan#Os_%`WD5 zLcZ{z?AbX*(*3F`iaIovohb2pa?acYdpGx{bB|0Q$y1|_DCO0s3QY>LJAbxZgQhRY zJNH)R=Gc50g2!!&*!LtxDK==Nfk%tO#$;2{Hzo)pI;+KyW#L|Ss29D5XC$; z0nciQh-Y%sP7YdK^vS1WbHCEW;3cZ_R662?`M}_rFql?3W(e6e9C?|qgwj>;FVW0Y z;d1EjAKg@bsD&+P<<5G#0ALVrL&19rFVNt7aQZ&WZK}*v z|IU(RDI_I3?jV$nxXw9uS5U5U_V;tyJpoHBN1Q+3ptnoJ((L%0|`bUwC>?}BYMvb6SEtj$|Bsom+lm5->tuxC1ss$=H^zl&+W_t)NAdRtY%g{sr2 zYg-K1iDHlBC`bCukKPqsT{lIGa|oGx^C1ZV>OQ`W$dZZ0CX12cn4>9g43{xb=`FLe z=ek*vqmZ6@W#t8MlU+Y%AMQ~Ry9YGgzZ%6)!zZrUZ{fq%r3-*3FS2%;3mr)lqpveC zW8_NaQ=)o76)2gfhO7mAx+$NGtez7k_xjL{2;n$#;MD#>`NY=lj@B^5$8V2GejOXY z&Lx+V%K)zI``frls_&PN6o1_XY?(gEr{%A#eryk>(|=S<67Wf8(IZbOXR+$(ADs(N zS*~^s@^h&Ut!Q~O@l!-~@Y(pJS{kI?HlX8Mnxgw_!`0R{trxtWMgrkzrF`X+SfBZ_ zZ#{Vl=zD|2vqTI+(Z|1QC?uRFrajl}Ra1f#wFc@fco6TaA-C=(xo_K9t_v|T)Cq|7 zDV;59LkXfk1gCDp<9K`0T%?F!oVlr{59GkLEO&=9EXPUTJp5Cm3Mi9-z)KBXtvnuU zxayDg#d>(Of}{nf zRMplgXsU_Ig2&wE+JGAugDY{n^-6rI9buPld@a$f*n4BRAFg24Oc*cSDao5jjj*aN z9~SO70S@%N?RTpi&@_*oXlmNcL3)CP8KB~Gq|n3kt=AF1PpxlT-aJM;Wmro^I(2Z; z>btQ_#`cW-GmDmfMx}S9_+UB`A;NuIRLb6y`y-9^Ux&O*__0`#ig=)`#igzs6d%xr z&QGQH#fYIGzkMYiF@A?3$Kk2@QR79?uXxWZ~fqOD}#}ml-5Czv5OxTdDAvH5opbo*|QO4o@1w{ zyRTof-9;UEiwFtc%+qik>%IEp(%`%r@M)W|2>zJa?6)!-o0t^KN`gqS?YhuVZPst| z<8NdZD^7t}G?4W+iL znKy0odNmh)^KdMo$HHw4LuIBuZuNv=;MWk0HSsa2cKoJWP8q^nHB;gl$hGNb`?f!s z^^&|H;6Nb8c;WQ-+_JiIq^NOK?KLAoyYQmqqvb-QyKV;$W&Q8MohYv^4;QlZD^Frd zRAVaYBYTS^;@mdcma|;B@=*eWh$1HG7n15mg5>ySV$q(7YMAqF+`E3_T{30tm*d8C3}As8>XNpMxIyvSHg7T7D* z`(C}kNiFXaF^i?v9^cWGjOo&eKYN^5>Bx!!cXzzv?uCzy5k7@CV_DuoF&8!e)C*6+ zWB-NO8uP+0d+R9u@BPPU&mmB@&rCihCW3owsM=GE(jsg0(RZe3ZvG zpc@OaFaGEgGg3ExE4Fgr&CuTm)mqlfTC5ID@#?FDuCBCO_N4mFmc!FX2a%13 zqGXX2 zz7W8jviZKWi>q*tDf&9H9um((eUqguIXRzdMcDU%3+F9zffA-bPO_2!Q#gN;_GF2# z-D0LN>nZnGi)HNtUM6ok-mqFV!y*`}FJl95lA2(VYqBLIQg3znRk_ay_P}_di~xU= zTr=X-?+Woe#nIkdqMUzcc)fZTj}AHRNr}PLr-)KXxPFqeVT>d`m%*-n5g45Mi%sH~ zX+7CeTqnkM_4e*BbV!rEN*)KcpYO%(S}su9zf|Kq{+8-NGtx*5&L4klDD@TDmAh8DS?T&=8(u;Cek76*-nwJ*Xa z{g?}lL-f}*QG0X9%5laFm>s$@*|lzY_tUzm->`8Ov8vWaFx&-rORCq|h+wIWS39PR zI-n{|PmcpXg4C9Tvue(^TIKh#k&scDADx>&=_>kT#C$r|O=E;`u@y;Cf*iqHbbR9t zGLA4I{aoxtI2A=1Fe3Z2rd*f~O=jfseG@eC9+ptUZ9ya@Ihb6Mcgx3j7Mzox?NCU{ zb?@iS0N~EIFD7BPboRmX&i3d8_hvK$^?3bfTEFlq23M(_*s22|3qJ%Cg7$oeUJ{S; znt4c_GARcdD6?j_V-XzT6|A3GTf(H%558bi0)j;mlt z*ZVg5VBRpMM^^*qp4N%DSoDRX6Tugc-_H;HmVyB4oIWzG``FjyelH%`kA33$XrVV7 zDWa3XmK&wt?`_F2>D+o*DBE(~LsV`xJj3=-=ZZ56RTnUNA@|!s&wYQYI#}M7e#px`mMY=l%EI70HT;KffSqXq2KUjCIA%j0k3#jX6=R2bY1S1-6 zFV4uUp7n1-q{dynLha3QLsd&iDgR>}A63d|Z-U>;YRN6mGJ){*s*vil*xS|q*w$_) zpG4v(_X*8I!+}~t9n{saX{i|oA5-ay>|^j%Gw0{4t5mkRtAtOvaI1l0sX3ZA(^qW& z{PZSwhh74lJHl7up(hZ<{9WbvtYkWvjWk{-|6xwGdr5_RMpGSFOkIEXmcE32IeF-i zC)%h35e+yDeAllUXg!Vly^67akl$i&*0ug)v7e(aT${^TN)O^SrFfORS-2D{ZJYg* z*`BuyEYNYG?H?6LmkSU(idvZC?o@+SH;dX@puev~V0=dAlR|>?==z*Hd()!qMscC-t#(p264` zVu4!1)Gph`TbAbO=oN{luANm`>vyYlEbvX4t_Ih6lVr;Ghv-P8(sVU@KOW#pu zO57pQ!`crKqk~<8amU||TvFLepo*!HW~!R2nKUA=SB!*Rxu;Wz;t7F6-oB_cQFn1Gwr1vo}`3c_u2>7)i z=iL==F=y%RNVC_EuM9mbbp0y#Y9$I2(Jn`d38+nu@byili)+@)a=$&FqC%&=;F%f4 zQ6zlIlUd1d!R@zjG0bc^D`vLy`l2vYB=%-e`|ZGRR(>3F(#2!wIE}HclKxMA4#vdW z|QaxxrFs~Qv_L2#c1{w!L0Y*`_R*}S#G2A2I|}z14(S^ z7>6vn z`4dA6*~sx|PJ09U{SRJpOC5+5lD$Sr1FvhX))`(~%w#O5o=UF!nZ1@0EXo<9B<c1Et)}0@iVUNomH+l1EC*9?$7K z!*ti4Kq03^S=ZfC0=l<+sEggB*Gn@sTa8s&f@#02cbHtC*}6?I492=sw`z9ytTYEK z#cFK@F@NgaUt0EUr*DlN*FSzHOgvF(EZ9VsQ{koWteG1)JFW!M$34rOZR5bz^1d>u z>nPHYTF;?32dsrW9O9dNEcs>T|H64oVfc5ErY#M-xibtZck?<$KHPrfRN%WbY3 z_8o1)yUJCa-9uk24_38hdE1VdWWX2R%9#3!teivDFust8eNp3%nsEw9Px?`6z_Fw- z5G_UO!TBEoLO_Ph`RvUOM27BOUhIpEz<1_BEvo)RcORY;o=I|;n%>e8-RXq~rf7qf zexn?kBxKRwC-*wZIpZ+DN+RsI7;fenw%Yyt%t84Mevh7^lEE%c>OhO>ls1eKrN_yB zm=A}UlfoFVe?-*_b(+v_3|*;D!YmB}hsjIBhs?n}yNVy$Tg5H-Novt(+%#jzNT z{Q~WJj{x=!YXyT`2Bpt<6jo@qCQ!z4&A54EVs;{hHb#dF>h{{dZ#B}X98@=T4DNS; zYDU-Gyst~>SJCdxs=zrBG*0~m-`Nqc!2yH}-hq@Ho6P3A;iFWwA>f5>1Ti3;wPN@?OJ3c$G{6~kNvby{+Eyc`@&if z;r~bP^`%lIjx=<-y|A=Ir^wZR4=|)h9(>MXqaPmp@7vJdKLEN5eK0HBI~!}CPy=3S za|V+4KBfLN@&h>aAc?x~^;6>gGgB7y$H7ZpZYwf+70Ealp&!Mwd~yP)i32ms_M`pz zk9?@LqWnLEodr}>Z@2JC0g+a^I|QUVq(eYjS_zS%hVD{QQd(NNyN6OrKtQRXyJ6^t zf$#9X_r2=>yLa8SX0Z;iVCFn$KRbVWKe*_|um(wF^szMH4F>1%vHsULK!2nrN$u`# z3?eAB3|RNvOZ#_U%>O8DjIojoxZXhjeV&^4GQrmU92JnYH5t!)jiPv9@JziktPYxg z&kPTK=3#;XDSE4W2o`_^Cz9*Tn2Gdj->Em>gJZ=&bx3vkzMa=GSr`Rm49r?yxBv9u zkM!W5+XVP1!AzCHNMG+!PtTc|`NVR71+^fZ85@fi2PQ`psQGMb7FAX8KPdFMAC7T9 z9;o&d7$4$ub&nOjfY?$$qeKO**iX90kG}n1S4}b!Qw!9=n zo?kJIpS?a1;Ftv1c7O#R&V`>ZNVqsVM<|h{#UliW19GSK?>Tb+y=DKn$QnYG+#Fiz*UPoouBYWFSfJ$# zG3^yFr9e5HkX;G5Nmt?R!km$AB9pZF%ON*H?G=wgY=k=iE>JPKStbhi!TVl@0ysbC z^UeTRkf+SilzWA5b+p=dKZy#4!z=&#?0)Q!I@9|qyxg+1eSlnNf8Ub;2RILu(3~#t zcrTToHn?ehm|i1ymiwX|CCqKizBDm`kEJkLLwt__3YgJ`csB_&=+!FK9u6{Ia@P1>3+Xk$TiRYS zW}b{(eG={S`$gXbXaN=oH zW)rQ!xnYb)iJEf5{bqadB}g2KJ}xltteD5_U>7VtJ9r=Zj-cd;ZC4oC6w~PF{fc;C84F)`5uKb!hE}5`(t)}Ex8C{n(<<70!~g& zHrqNM$`EXd*u6Q(4kP+RqukZWFV4-c>^Hzr4(oGn@^%GV+Y#NBKx_(O1qFprRd(3Q z3SEhA#l%Y15ZKVrndz1~fSz6wQ4r&eBlCZr*1wHtO`W0 zlilW3Jh<*Z*C(-J_HO92$A^xJ}_Lw*s{I;5 zFc~qI*_Pte<9Z#!$ii%y#fXWf+5Zyfe=Usy845nz4XH4$q>qmeYSHH^N%z4d&P<`F z0`~LqqXlZpXS-7t`}3_jUO;D3DJ;;&W~y4&O~PqCLKJ8|pi~ocTn&vwNMqA2H~832 zY;(LmIFa1yd0Jg>BoIX@R-F0(o4jz}8hFt7cz85G-`OurY$?fLGZMb{B=j+{fDjkZ z+Zu=emxTR!hp}Oj8#Iu#u7zIh$c0bo%3;wBITRsK|s(H2(ro5 zEpxhv8-K_epwSb6foI=$r)t)KpH@x}B(huErbZSPGS?JAi8zg&r+cH?+uI5C;WsxY zlU2f9yp8#WL!anE3d&VsXJgR8Q~~Ju`WVeywDXAXaZJ&w$dRO-V7gAO7SryB0W&04 z*(!5pqY~@RN{<`gDwZ)#u)Mk#2X4?&Oc+@dSh-qsyFE0C%Tk##D zgw^pY$|7|}=3C@>GqEH)LAshdp(0DH-eqaN>K#}oQ?~g*Alwu3dhf;2bX-x6@s`-x zA-LI1D1by+?fcOkZK%feixu%sbSjs5D6b#@*!t#~HkTehA0OlXLlst`Ra zWj!JL625EHr1D#-^eJU2Cvo%%`+_LQSwaX5qy>rsO+;n@a!#!vO3vW!7VZK!%9R%P zy=i=rA*zQ>Dax!c{3zrZQk_3wY86{;i419yr_F#tQD;W2SR2o_;Zo#sD#p1M{D`_7 zD+q}{LaQt5pl(Kwox6D0tgfvwA?5T=+sk*4&3npczsVH zzuOjI#v>+RPag*qwt|hy+;>oRNn z=6Q+K^sg2$%<57VV=>Gsn85nvY<=!iwZ%Cq#!#$!{}Zg3 zYLHca+9>Y#x$d-t0u;_I>i(by3ZugUTqyq4;9i6zCBQ#_H5%CIB7S)*W}V*4LiiFt|=D}BL7U^~}b_lQc|P{c$n zU6|o~!LPj*G_JG<*%$W!x*1$--_YGIljq6?e;|6JMihAk0hC_Cbe;78j>JutQD<`; ztT&vq?cMkqY`zsj_)a;VnPvxCK_uvUzQ4f9XFW0IRE615PqRF0^D`@8bvV-`d;+)r?ISgNaF_Gv@_>MF$9*`vczMsn+5s1 zzMC%z4DveXNT{C}B?71{sXj4~4Q*qNYan5xF0NV*vAHu=>R`kwEZHm<1q&r)k+g3} zr+?I|+r*%G!gZagssA%j5TYBFu%7pb+&a5!=zWuS6$xA0l|yFfEIB&@v?7f!@j}FH zoMkJ^*ip#qHf6Ml%q=4cFHWW@il=YCb)egkDpS}^`wHGNER6OIbgmAuxS$VV-c=Q> zR|QXXU^47u%@*xY&frkI4Ellt z>-BV1iwaT+-xk-^a^qJ0@H{%X2;8UT0c|AQf~!Tv+9jaF$F!^>* zQ=p}VNdkTncXJ9g;kFu~Z$&{x-Kzb$>F?R)bA9F_%tRxiz$Iq>9T)aeF8vL92d)d? zMZ6aJxU|>Zf>+Ug+w6U@6rzv90pPh~CAw;=X@WF>D?HK*d|`Fj9{V+O^BjpqwnH7A zIj{WtP7DIZpRZRaLmBO?WzDwrT?;otKIUfvPb~INrnzA|AGuroWT(WxBtG)4Xf!_x zyWJNF{J0@LSE{M1hn%;J)tGLHU0RrQyZ>0Z+pXV#E}#o<@=KSDmt^t_g~qdmrn~pz zt!vUv>A|1dE&@l}rk;x}X&YpBo79TUQ5Oj1LKT+gs|OgF4h3ZL>W9p=$nUd?O!+3}b@mv&Jv20Q^DY|jd1Ns=U3Mm1wljaG16{IP zUPZ5}vL!2nuZiqT$DYVdJA(NFJOCr{>fsYsdWGj=Z@!_VACa7+9=Zd4nWkz^wnHl^ zD6w(=D^J2#TORf@{^7>~jtf|Mv4rYeKU?AOy)Sp3eb6gAja*HiDn;V}x(c66TTgPq z+r_jJE!vtH2Te`7y;Lg=%~7U)FGUcNh*lzk*W;nozBaD`7_CRTQ7gD$c}k!{h=vW3(qf1gQaqX0Pr*W91`pnP)4-c zuVj4t2;JS-%08+RW|66+&?^%-xyEw`FcGE}>2@3!OEpyq!P8ULx4(vV-Z%L$ z>~_ee6HccQWo4LkigOfc%vfqhw8Yb01i#R8mYS!ils;oF;QR2~m57a%R1)+R8zdT< zrxf$tM71-{l?hJ9ba?FIr_OT@wEM?4Ic+FKlJb?lS6;2K{sf`|4eaH1b3#dvEw{CbPOt z#tecniHp*FyiJo3JdXzj>If!(pmLbb@}QVKz?j8AdhEZC;$j^4`$GaM#-}8CZLEFB@VQ`nSz~yN7$>)P} zE+0`hS1ONX#1II0vJ1JHV#_X5r@Ys!AZJxA;Ts`T4rZR4;G+>HWn)(RX>B6ygX*3`{C;1;lTFw(yZTlb=<@j&O*{$ zQKCH*Z%OCM%o`fwW7xkPjwNdHw$M^0fxl$@csXS#_}Og*-N@ddw<`PxHVQOEi~U61 zNLj?Fm;Agoqd?>3T~Wqo#+TO@>6Q;u04BXeM_6GUyZk9mOT8*~k!SaX$eZ4;kkyLk z3AH<&+vGI?P0T!@-q5CaU-6mfhD+}n!&btwk2%A+75&Pm+`sl7pL9B*4xf-0pMj5~ z?xxh%_>z5GF|a7)bY;^DBi677dj-S%0^fyf76 zNYry)yd{}0iYJVBk^T{F?>kpP@m}G}^a~786^Tjav-sL%%=t#j!mBU!`FOHmVNmcb z_eLS1K22L=I2}{YYTVM&)yuk(&*>DoOA9vJ%ggO6*rujCq4s0 zKD>dm&}+MfE(@qN9iO$;8b{i6#d~f0rWv;-$F<@Q>_eBTVRNBr#f&vZ2*Jc~)w^_V z%RFgkDTyB#pV($7NlxBixaci@>t^_k+TUS+06lf3Sw&}E$jO(( zFvl+eBR98co<-X8Q{z{r(PjFQ0gKytC|TP|PXW_GS)|WeGcQlkFK%{T{J6xx%n0;F z$>QoegzV`d5%#MwRNL(J1a>Fq;t6eoA;s1+Ywgy7cZk9h)WwKfFMvRSbi0acQ!GofomJvT!v9SijqDM$&jNjeTGw)qxU4=j9La z??_`a_(ncNY$^YO)Y?X%T_PkQ0pYrQN&kA1ue-kWE>%sW3p`5cv6Vuu!&#o(QNvN* zQ4{d|w?iF9gRmsPV12{*HcB)EMPYFW7wkKDKJ+6%m*J1yNgcqe*ula2xF3cqfsc$# z`=d0Ow-kH~pMa!|J>3D1acJ&U>tx!EWIxB^9s*d`ywYV`fob5>!`7FOi<5O4i*xc+;#%dfqzDb~6=hneLJ z*z&$wW|Bj(!9!g$BL>%f2~WM-OHQe|b3Tmk@z39$0RF((Zf9iwDKTu4ENd?>n7xio zxTDS*6se;zw2I-=zcrn;?s^ryz|`8a`wIeoBB9R)K|z(lgyrK*_&pWO*RU?o9O}bO zXBCmx8LOy1U*yLKjekjkvWtj#sUlsa3Vj*F87ad{Gj zMsa9b)@6GD5 z>XtuMDEjfQT#_zQ)WQ$LXmI9V2L3+Lk*qb;s36XeGOv0v8~bp#$6bWqeundG3b#%M=Xegku?%afzGoj8^cJ78TpV-+cql1ZBy1*46ZDBmls@3kyYQisb3J9 z7DJpkOqyttvQ6i~>J;218)Id(KZg%PwA7sXk~*P^2mEHs7Q|lW4s_--(P)L1)PMMS z_(}+TRxG!S@ub&rlG8Mc7mCo6&o2A2Q%vz#oD>n zzEXJ-u0#sy(?FKkwaykhdW`P_k9%HHB!sqXkCW}Z0&;MC8mkitjGmCd-s~iEOHX2D zt*f=pDu1h*G~IPJR$@!(WCA64Zyk>2)W5m)>s!%I9u%$8R}R>=Y?RJ zqJ%v6#lu&xuo}s*k(}N2TlYt;%`^0>JQr*r?Qu(~G36^bWUID7Mzt2I_HHo`BezbV zsQjGDDc90ffN;BrN@#79V7q56{8)4_4@_z4tAGZxdKBTbNO=65>cMZv8&w@YkS1dPif1;hSvlQ%qK@5`U0lBr zOtIlIIMmud{!rbtfL+D#OfltVCX5|WBy z4H#?b@E2DT_|C93BILJl(87f%00tA-s^zTFqt_e~Ux$2?V2l(%F z!oQMsv4N6d->@BUk)C^J@2hz+ITAZvIK|R z+{$&;<_t7-y=6hC{S( z&C1XWdH0gN1RtEUqeV9-Vw`rNnQjl{cB<01^B!I+J~++(7-pi?44m3wvy50Y=9y=l z;&uboY>;Wt)!BlnJk@gf+OX%a-z-lhO7`O7GCx||Y>NE2HW=5qs&U04MO_nTD*|)x zib}#qmXTd_Zx46o9}A;E{p-HyLrS4W|CjZgp?f*bsPN;Jr$$sE@0W&;Pg2QMt5Vk9 zJD+FQ^Op21ty`rb_mZow{3LcW4prF{%>u1>KD=&nwajd7Z%92U4YJ`AiCmfTjhd2! z&BlL}020e1Z$jAA*Yl}u`M2@yQ^A?Lnd-WPk4k9MjhUb7A?0+J2M+aJ9v^tCoei_r z`g>K8@CB04<5D4AWK4C>&@okfr`xxQ5n@y8OCV+{CPA~xlvfdk6N+c+u(^7^Q?9bZQbk7kSJLjZuH2y)x~&MuQ{U&R z&%ax=jzxE}?|9mIpD%be`=>;&h|JaZ>a3wh!QnUSP`K*Q+p(Q`D00;Pskn{N_wA1x z+ahqc9{d10+KT?;Ok17IXpJ$Fe@bFZAd1G>lWY1k!4TTN7S=0W^7z`RUy^~}X3FO| z=BLTMYaOoWhM@RzSHz1pVnji;%4mG%7H3i@j5YnZjV`H+qd5h!Q`>W8E`(#tC^Tcr zv?~Um)%K(`S=A}wmDY^(^rS}#IiCYTEdF@Lv`1esPjTYA%FnT*p5?aQ1;~AsJoa@f z#%=C&S}&=l_&8*RMwXi?!>!xhX+mnqeu(?!70&EAX{Wy-=oT9z=Id=@47k9(D0F#~ z&~p>}L!S`#BQmswaoD?Bv4AcZxxUbIYQ;qR*lWN}J{`of2N`}Ge>aysf5KB-e&ANz z_Uv0|`=skR{JBW>%5l(YLsPF-en5$Et7lFF|92;xjYL+EIBq5pi@@IUspSa|E*!H^ zztxs<@WUW|WR`!#2rQ5GXvlx_2ZqA<{Nm~9NUdF^k`Yy|NrztUdcY;%4eksUJWQTL zDI?PvF1Mzwn08dnL_6mppNh18OO%@<$9nsDK=s{Mvd=Y3#c%&i&fQ;2YT$wIK=vAu zLk|Dg1)}&0nd0du9%5s{x2;M9SA7T17*M)&kX%cI+wIR6-kb)7zOI1I)Y@VD>;`im zpOQ2POb`>-K$FE1zx#WBVFc#ip;xcGrA&m&=g68mk?&aCWh7i$XUl5la+I4~PSnW` zUJ@_FKY&}EsaJ`FsbuO6sBH^>ek;)ggym8r=Ze)>W{ZuOR`V&g)zRCSg*00sZx-nDV#WOFb2x<@I` zuHPaGfD`f-am8+t8-W-tkGQ|630=O(i)qu=dP+%LLTTrDsvE>a6_I{B6@j{R3N^MU z+ecaFcI|oNQL})3c`#!%ZEFp}-OyR+wt}|t{L#X zFf#tX;<6nE{jC5=Lub{ezq||uY8sU7BCUd9e#b_&5EW5o99G@wjfD`*k;iHV-iCQ{ zSV^8RsA8kz8BI)&^G~#)_g^=ZTDVSu6X|_k81J>JOxTGqfvd`=NNG5WoeK)5!u%J1 z*4g5SZg*p+HE3Y&NZ0<JnDBP=<7XFU#SWM@y(j%%k$5TwY7-T*lqcmM)Jxx$El-eo(7OS*S)5&s>Tm;eJ zi)b6SG{$_P$k3m4-uvr-T>2PQAw<&|%-Exccpff*_%_mt~^3y-;ItW9{O)U-xOR05`^-Z93k=f9&8|57~^prB* zhZq&SKjVEc3!xKEh+q)Rh-$n~vnM8`Qt$=OMILH>NaMa)(O6zv19wckgXqVoJ~`aK zAk}#0w(D6RqH36!XvDwe4BhKF)rgudAX|?&(Jp?~C133OHi`I?r{<(b5%f*ET}Z0k z-eSKenM5`JtK0@3?YR*;zlqVAGb#}+rQH9%6j%!q_C%(-Y4GX7SFIfnD>hah8_5?>fhq>R5%Un~5%e zEY*XaLt#rF`UXIA*1(auvB(@2F3$iBm#;virrbNtyJ+hSe zx9;q?`jY5`#j1Sv3eg@FkOs;>Er_SK!O7*_n-`aP*&xUjU_C^pp(l(z&!9<|fh!oG zA{kFvE@KtiA&0=<8*6P;|IS}Y*+5mCvRWBKGRpa9qj8^<@5uG(%k*u+<+JXAU$lpj zWY`&CZadu#IPXrV3+GlB%}2Ae`HG-^>HfLv!=L*w6)qQ0T^>c3PLwUMCZT--?Usw?78$pmre&>i&i3jdnK?;j4(Qr#j?HFl2W7Oot3Wb` zfudF3_=euIe7S?N^`yxaX9T81!=p4eCW?+eCwcNyV@64|gVviZhdtmSAJRyofxVfl zaQ+p&NP?JvZ$Wo~9)5p1KPwDup=mX>S4?M z&K~^v=6_ws5F-JYK|*0+VOLte6JYg65D*`BSS?Dgv&a5jz5A!LfPj%piG=*%QA|t> zE{Ul7v!KPX4QkXKI>sWC9j6~0vo zKK^H*`TyT)Oa}%1>%c|M_8Sdhmu-63Oal=WRi=6Rul2#ywVz+4C-`pQ@I2LY;fXdM zGx*i1<8l{935)easYa`(i}jt|%w&^O0f4#4tCcve^~ci5M?q{qrr}_8(E|JwAwK{A z@Q(jj{f<@@YKEXiMC1qGC#%fZUz2c~#{mCE0SvAj4+cz$-FLHSF-G}l%DU;Di840j z#HUqf1l3i}TN-)|cIo1IdNpq~auq%WrcqY3<;X?G`ogca+JN~?F~BLjT|1&)1(lW0 zJ_|YJ+r6JOD$!xTeq$FW8B6f`;U>L&KjQCN=6|H@&!>2Qy|_mWtk1tip;AUxmf^aq z+z7%L$EfnrCH)DTj$l}Ol`wrJ;`?3Tysv^Jc7uBQgWdR@+XcTnz2h`7FV}vfWL>42 zdo*b(IzvG9{}SC89rWB<+LFz~rfsLiLtu%u%TK?oA>oj_vL3E#S<6H_#;!oQX2%G0&p&EiQQN=MD2!)3>qTyq`G& z0Hbtam)JAdjP(-NSL8whfJCtzM<~~;$uBi*(!k+)_mkQK;FM4JyA0{v#eqrb`tQ0C zp`;xEniK~BoBdy;Lry%_<$<#%Ehb9!G_qwdK1=wu@!CxOBzM%$Av^?F7UsK~<7i&n zX)V=EahNl?$n)pV0ZseTdKtg~69L}r6_kbF7B{@==4{5kxz8B6(SD(ABu|+r;QJ0K zu&yIcgK0uF4EhjYwDy+|5yV_l3md@UYfo`Ajp%%?PPoB~k{;Kv1|uAYOq6AMK+UJQ zyT|;cv@|#a2rSFL`6ql`!P5W@|FYUR`a_(gKJ~#&OOE61`xP9W!R@t}w5gVrmeWEufz{qA`BHIh zr3XrDv7gJ}B2`tUqk2>1zRyjBnW;aB&OL*sCqJlA{JoFP=>u|IQSPS-RfEP^VFy8@ z@Ho1 zFSN{{ASA$msC)`osvY58Vl-7#dwYBKT`BKi=0{6}FkE{swJreDUCoIWADL-uYtt9( z!325(VBbpd7HK4{5cn%0ik0RVqCpk~fEmk9uy;()^F4Z*+uOMgnT(u4Vr zQX-Of-E%yzV&keohCroY^%fckFkt#{Czb%_Jk&)$e%V)0PMwZuBrxlZK%aq<2a-v| z4BK_IaXrV)u%D%cO#}Bt8zfR$9h$eYXf*K!NMyCGARU}AFqLjdE400M>|dr9LiXF5 z;`$-Up=UgM^Zo`nUY;_&q=6J~*#;2uT0zXOr;A@d!~(;{K8t!(i$9@dH~hA@3=B>C zxoi*UZySKQb6orATeY8L;`<3$aVvp+z^1q;fd$P(Tz8q}5PUD!lImVZIMD2__r=L` zD&)-O$m943g?6pa3xl22R8JbwS~;dXYc;(Q{qP0sJFP#7%WSl}fO|yf4ZdJrz7pxATT&r{Cq~_q;-V`1ogbl>>w>h+pP- zAc=DT7@4~B=`&@hghEG<|8@Irqw_My-}X{+n+bej9)Cgt(Yt@Ow-_`_RKi7udDd1docx zUUOFotRny!iHJ)>UOJcGDu2a=TjY~J8UTEV`M=@$>*?gmB5ki(dhDmBbK8S06f?z* z^kq7{7z23KUN3waI657^UL8%68WF`r`frn|Adb}Pb*}nzaW|9N@Nheg{98rSGYvty zaV`zT3IH^N)^=BEQ`1yYHE!#rrwnQgRAIvKGpGsB9LUzIiZ zyG%fdn6Xz0)k+gTNK^z0PS@C=7JL1pM@ho_2Vu-j&X>)r*k_*}b|dAUc6!hk6uSYn zfm*rO2Z6}VQFa)o6z0G6mZT%@CZJK@2V2~sEPbqTa(vi+hA9E`HR&VBkr|ewtL>SP zvjNy0Al2#sD#i+E%1;?$Ud2Rlqbo5{08b4nU>{+Ggagc`!Jx$5O%0*nuLHo)^{+%r z*ikr@ZG>NSF}Cu82_#G`yMAsd@>_bJKM{&j+V zNUJM6_~hCu^MtD1q#D!su}Gw?gPwqFlhy0eX@!n|L7jC+x-mtm``v}eRLoir2W z%dK*9`NTUS8nXmqxXsEe{wkH5@HDEmn)0>y%&JzGwzdBnkU0c+-dNGE^GC|Vjv7^s zPN1DiWTHITjXRVmd1!LmYM}RJr!mQygQC^pYYu4VEK`7%e3s2G1o(*DYTl0H>9mT6 z-=ga0Q1~%JY1uKjuXHDd9Y=2+b43HuW#LA6bm00h{czF7T0ljMh2bWJ zrSq$Gp!Vn)a0W^8;4ACUcHGD%Puc$Uinc2yV~@2b&*2NefxiR5Y(q{3mN@{x-2%W} z@r8kbL%%+t(*<2a5P-i8fSAN0(w-;+xMsUP-*1R|gM8=LISGUgj|ClpA&)L3!p?;k zS6ACjKQbj2iX>crTofkk!eH)Jljxo0Op_Z;ufHO(lWz1U43MJ1VaKz8YamJfHb&C) zo~M5*^pA99QA7u_c1>SgzpaSXl7o=jQnND&Ax91*G3wl?ZPKjdhXxYXH|4zOat-d6 zN^EF*Kys$~O|M5&eqa^=g7nvm5t;ueut@8HpEV$JuSO&57%YjlIhtHPb$pJYj_cr; z?xckTCvTrLpfKZSWEW(tU+(2f1B-l{lNhh}#xs7?#fobSnF0-sI<2=2Nui-F ztm(mlefGyuLKG67+aGTl*fiQwy%m}pqD_jo<18ySH@dQ2EYo_vVN%D9JK+pRaV|V^ zNrfXw@sF(B@m`0HHcfstJczh9yLBki5w5v8l=Z-PxR_~R8eoe2pMBRsK^HRuXk_(9 zvri)zfEW8yMY~w*lWF(I&z*Kqg|Zu{iC6d+9(9Dmlo`z1xn2cj>eXmMOq8a1k*qoZR0LdTINC(Cq45~PDhhZXai zi$ndEU;{xQnw|b+BZ0R9;N{v4y2VVPoC@+3*>}p)XuVS> z$h>oy&*D{Y>`|wsh56~^wo>vS?FOgth=*SKII#1rkNOV4?6+?L250?IF@4;%PR#8- zVUk398U0Fgf{wj&xZ%g z9M4ohQ0fTY-L-p9f41JMnQ?=x)edUgPutfSS1(13?Nm-E-UM~%SI1;tm^H4} z>~GFI84jB7qUy9c6B|FayOZPwlA^6Fb>n;A-RUoZ-5XQMx;C3(&|=GI>6=0PN&0=~M?>1qpQSZUD8`L`fU;U^~qYN9m2G3<#p zQ`sil-9msiskb$NP4Jn$j=c7{55mMU#k_PFUnEllGVlp730jZ{m4z~chV4P%9}ED- zJy*XDG>sR*JBDjy&|p`lsh)bvI5Tjt)ae2ZIvy(s~@I-6^2Aq#YmAtflPS5YU*5zhtt-02~qRrvt!@c`Q?#2^9=|?EYv$2 zpe0|9T<<8FO`?B!t=_W!YOlRJJm+E`T1FTt@Es`+miCxllgnj<&g7L#wz^5F>Sbn1!=cf)1!=eKJx)k&1j%n+^f1WPYGhAoG#U`@(%MaywswbwyKAESg$Ep{z=dL+{+0NqJqRn3I3X8ICwNRMNZ9Ab^_`e#l*+t~ZP+7%oB9WZ8W1!&N;0S-!{70SC?oix0{7uz8bwz6l?df<%;wq|M_T}j~6@diZLEo zG1d7e(D@YRgAQb_S}+8SA(7`SyMIO<_eqKrj9S}RUDlG5LG(~oxZgb022AS5RP*`! zy*PF!IS;A4Pp(920zWL8`~^ycQ*gC$Mb1rXo7*S5e2S-lo`w6OQkm~rWCBpZ_;vOQ z_WGK}`uec>m;LLUH$5YpyyH)lQS?>2#6ma~>xgtpQrf-DFa0d*sN;3>I(=G8iq%RIbm;z_yS z_ds>E1{)wAlJ0Ai%bZct@wdy7rl6>%s*)9J?Sa`YfW&su_pZ6?f(rr2rR`U}mb`s~ z?%Yx(k7&=XS3d^m*2&9%C;q4z&}Xrt7IItELLnCY@t^t=*bGdI3J#a-u(7cdfz+lo z-o(iVVnZO`$-X{qeVVeeH=%%GR0q;lzHx!});?j?Qn>F$VtJY6Kk72*0Odn5MQ_J# zf<@U#NJyj;k`{f5yx8&NI9CM9ngDPSNO=8M3?@lRjc~pru>VAtr9RvK450qmKJn`> zy0~30g>Y>1_6h2=WM$$z&)%gV{wjFqDDt=-%nKqRr`zIrYyc$8^Ww0YV;Xtxce)~Obu=%>a&e(ybIsUbuKHRf)z5 zOww@BFu*($dg_KJTQ$CnD_t@n?#c5Il`Ae9Gy3m?GNICC29s6$*^jQ^{&o9S$aVII z8lho{I6{VX`Gak~>og$<-iJq6B^>9f3FUK?z_znmaG5?kJ{@66O((S#nK9#joK>VS z|H4snoWiXkE29!FR4F8S)lJm$RHMPvx%pj0fN=YF;-42~rz2iJD7F1A%(J)Y&7S^2 zj?H2tK5TE0j^2os+-6+78(C$7#cme9noCp5QEGA}(@INWs|IrX;@^ ziDk_cj`o&*yGMKA{9K8lkG0m^LU-}oXCPmx7-7G|=GyIgJy+FImw~FmWWzW~5Vb!M zeb1eX3}LBQj{U|?=6PRSF}rfsh6KZ|n4@C3q05-#@#?IA_rsF?3b6wtjx6$Kh)vOw z1G%k(?AO`7=*)Ka!L6x7fVq0lAnw3FvUE!|S63^VUyYm`feN*sjUQJ+?|I*GTwr=A zX$1${rSDCP&@D7X?&F@4M97&WL_YD*67b4r-$V;u?hfrU~Bdl_K!!fQ6G#-06MEDqX7UXY@ z^sNLU*}52Hon!6ZOo=l{<#Vf&Cj+^gV$ck`HZN%`Ba{?E{>7wiDc5QIa{3Iqc?ti`|f}vdI|vndEto{ z%9RIU_^)9;>yD}9$1oxeqlRFNmkX9^MQbVr{>hdxQvy4qc-VMA_QK7!ooMXRNW8e% z``Xu`sUoddx>RFr26;FWOJryOZDB7~xS}*c6EErq%9-zo-qqf1vFujB_f- z_89Xg9866|;iKx^7{`ykh#HAwKomP~cD_k=>vd}NxtJT-{o_xhC?esXXytbg`$=|K z&ZwM*NMF7XI>P(@=ryfI*jI%o`z%UAne|z z9%cHo?~4h>(?Cqj2_~dd5gmEXIS-c8vHOUd8zZ-?u~lNC?$XP2UiMldedZtgk3Pl} zWfMV_YpezLFTFLJigMLv^n&}c3trN>Q4#n`wGG3MDlRGEqDm)EqwTSa5e7Zp5ztzY z_%86I%4mkL>h)Mjy!Lk27*y2p=%eV((1XwI>$Qa5-{NCN&I?){@@;JMG|CgP4xuYF z7N=}LKbae6c`Hac(3!G%0N?Kz#cNv>{rroja{J3LY<=rRvsrYO>ACepY*)LXndeGKM@>Pm5U9g@J= z`un-Ya}D45k5h0a8@B9%xeU-6|3Lizy?!i`)O+pAxg99(7=P{2fAI;=wY`p4_=8r` z%&{3R9%);}sjzPLVMY%~+m|P`L~|s_+1pm4yb`S3?}&+7kmK)YAoCi zx3?XT?otskx3Aih5w2}1*a^zQaLMhO{%|4^$>87SW&-f!dQK)=}b+W=9dLAtxU>yCwcZ`u2t z_wMuIe!G6~2g~I@<{We6GoG<2adtz!#%-z%(s?>jn)02umr}lH6Q>Y5PQS`8@1caI z$A28&zz6^tk6-z*^4s-+Lrp_UU^|w&FIy&7(D1W;fktC}Ri-mRpwNVCBCYbGuzW(` zp@F3WvJdVcV31X&UW?2>g$}^XV}VBJ%|3Z)g@%=dQ2J?fp)xVuu-Vz_ED)Qw=e;Q- z#2w9Ek{8S}Ycc)Uht(8!E{$ZCq}azk*6!HK@5d*}l`5CADTo z?%jx5aovq%=~=C4GkKoTrWqhBtK~-_sr|?e2Ctd~QSGsQAEXb%AbQp*`+g%wnZEKC zS$}iM`&Py|jqFn0qxogW~Gu6&B)HUHYpznyA}VnY50UANYj z{JRPs?GcSGX2v0&;|gU|+AoL{1U%mkl$ubD3y9J7DWQ|l7TFoezxhC~^h3WR`IDp8 z2CjvhvW0Yu%whL6A8oMn+=uk)6Zx(-uGhDOX|HkS+ezG*_|ZTX+`3?xBIgqxovAP| zQYp$h!xaZ!a)^45Q$h;S>k$(qzEPNXy7fcKe(7j%B+t^d$+A^OVj>zgzG!I?DPQX$ zHdPyuMHj0mZpC}botxm4BSr0V)!yYuH?2J$|r2fFXLeW`o7G>UB z^1BP`mz{PEMuG(r#n!m~w$K6YSb9GKG;!&({Q9DC9Lb#Zqu@a*-z(CplA(U zY+lM=9~sz;hE0rPOwjOJaC1ei_A4Y~c=If~i9_>&_)@OJe4|ufOqIMac9+tG|4u92 z!m3L%4w+5sUSV9@>WBWHv=^7wZ;p7ca%DORkKYLqR9!EPZW1*ec4o@tIgzdSbpRJh zTuZb*R?Fp&TK(Jr9_JtmLpYIKYe>BdED zlG*h|c`NH9T<>6YrpkLpE|?gXSaZ+!*F=1#e-iWHP7D}l5`|HgpDWlMsDn;ejALxL zlpo*WE@G!9ExoQ+H+DN40AyS8H7I<4>(8!% z0A)0_%*F&YJ;`T(Y-H|+o5+4yhkYZwd!Q!v_2 zVx}WCsK9(k)6vl}(6{anU{h{ga{%p25;jeB9FnU9bKL_!mw&36e*;9zc&Cz9$ZTi& z+zh1a=fu*|J{;oo0hT zjFHy127!9hdS2Oo7>>`iOw~dnthVl|s2VFeI=V}My(l|eS@3z%q)}w@{>=HC!rtQt zds|(NNQy6VO`;KxzO8bG$wM}70G3TBuBE*grdjF=H`{~d;EYF zT0*hZ4E(1P@Sl$-`vz7Cs^@yk!yk!ANke4Aes*!%EJh*;S0041vK#>=o~Z@^i3V2( z9hSZ!nex=ex}~Yv%_0Y#_E)}MhRi7uH8m@eo9ptSU;(R4+^fr)qCiuY9OKL+a;?Ze zxiQUO@n6`VM7nz4hra-tO*JA_-t zZ?(G2cBJ-}Mgakt9gs}w0ij0T(^fi`EZzOncofr*SC({u&Z|uS8IU)u8@T3bj6oIq zrpuutk2FW*5~RdZHOh;i0;j4m@=Aos1U<_&ya@@+18jiO>Qr-fwo2*qCr*@BvBf4m zbjeatZF}3U&8RV#c}0JqZWY+B=#7g4>yS4BCn(GbC&Z8_T^ z-dk*L?iIDTYi9&UHA>;sIkeP!xmBipMLM!9@N%+`n8|VtnyC+p?_#B9p?)Myjn}fD zxoij1==HjD9%pH+^FrtBm-){0Fvrav12d{Rg>=Ob05)Qy%R9WQMs;5Ph`9>HHgiHC zxpCI0y2*fSDWjrW%{-J8;4o`X@|B`Btv28IU#uJfYa5RdkJ|3!!S8z4_! z?KkF!d+>M(YR-!bff@>Orx*PzA9GTv|Ltd2hd{(LbOr-o?GkQDlfI%x6QiwULKj+1 zsqow4xLHu{>%HNlT1_A{G7PXp8yuB5bm4%PqlvP-~kJDTVpcT>^{o)mR zi%jY#95L0d+SC1?EavEFcEfFVXOdZRZ8l~*dk=6tmTxt4(*r=W(waP?N+0j*iV!|e zLhTT6mdvaSJHGbBtAYgy0o5C^*o%rc#fHYJjMZ<8rglFV?y~A6QA0WNdH+i!7%NW&l&?CgfJKOUzb!q{(PF{x>o7)+B3$PGVNOc zacRqWEX2}1iZa;)1xbxL^0PWWs3IQ@uicL6i4}MWS!*_mhY@!rhxxbG)`DPmdxM-b z&xB$LyY927>u3*AqpoYtpP*Fxhmrp4*VJQ*MS|CBB#ib!Okn4<@d{r|NmcrCm8X2f zMI~tfFjgjS{U;#j+ZM6Q`056VR*4B|6+G;ycE5!b;31~?}pb1B;K}Bti3KLM|E8v zt@T!I5e_%@oWV*cs#CWx0V8N{Cia2*#pFXLN;6HYj|75*1kXn%AycrWGngXjQ^TDk z9h0R|P4IXm=QZ&;!5iugSUCd2nj`!7X-x0*jzo&o9PGD>UNn5R2W)&~C1ndTDUWDX zz-m@$B~+cB0}#M;S^c)-uwq>v5aunI)s?2aHK}9QVegeOE(IW(n|&)m+KLK2O@L6a zem^fN7s5P-R`hQo`QIE7Kv_p4nH(*h2EkFdlDdt1eY4<;2&C~eJ2GC=pDVOw)2hnY z8p$ifbq4@f`DL{Z$2tyWnvysXw%JC(SGPyFnvkqy_&o#FNo@V;7FV_w`jqbJ(WZsh zRK{LmQ~ySgOPe7$eLt8{PNa*;J;e*-Ye3ccl?Jxz={xGRS03T^cve4kn2NIbL0fFh z@rKqt0x@k2lh|)bV`W%~(+8MQxXS;iGsSDYpp*HY;nnk?(cT|&F(hlJtU`Ue@npyg zeI*eanV+ymr!kXn&kB#d5a~7Oz^M~l*-|7KbLn5nzH>O0!|-o@95{FUOhCp!j>a_G zMS*%F)x5F9`Gy%lRx?MyQD6a_c)OXFN#ARobUAMJVIQhn`aU!Rw~SoAk`g>FHXq7r z7exYq%*yGaZm8!fU#F)0M&SDvS|F#G+a3zFy!uRzy2Tcmvxug1L-=We?A^z+3*k;2 zreio9`fpczUpe(9IMv{+&;DZ)5{AhV?BTA&HwWUa5NChxT4H$H;RqBSXJ71U^=x%w9BDg zB~2n*+aAS7@_+!d^Jlft(dCE%Dy}bAZULoUorOO3VG{H+3-2zt@8cH?hF=RFF56UR(rz}*;7-;aii+;5%hzplBpIFR|m z$NMMNx2fisux8%9hr?!dI|QXj-xKLBK)=b{+G6#Qvo~s+!k9gb4Y01>on_VR!Vzn<)20e1&~H*- zq~q*0&4&>^JBH`@Tv4;TfQk2xN%;{QVQAi6W(`h|~VcOc4p$EY&SHC$rld1%u>Vd%O zR-Yspfu5yEtWQ~DMtB@D)ny%PMEQ=C&I@E@Nv|ckc)QJkB$CWD*O#Ig=@*v(>E8k~ zpuh_GJKSk}5Zk?AXDji#R-wgY0_H-833;5wo+Z>3h#g!;pq?r?rNVw`1s|&t5J#_Pns@HTy&GY5Q!ktGQAaLH*xHs3;t&AqA|~A|4v!$hNOH`WFSPTmljXXo5g;+MZQ=AqDSQ z(CyV*KtPi$8IffCSx~t!d3g|;BRKB^A2LH{x>E6GV_p4QdW9l8Xt2mW~p3pgu?l)ysVjgLvt= z?IPaG#V<5|R71m!qLJAx7sn(E^suWkg5}vOOOFAKqdM*0Kp}X&P3TsjJ3m**#RL7e zGqo~*2nmf9A<||45p|FKr{oCV6>!<#bT%D#8hZ9GSE=92ccl><_9GUXijs7j74GUP z8w(miAODp}Vix~DTy_*AX9|xPC&z+P;%DIOqqPWG<#-nVOuiI`;e$Wl{jQR4Y#PxYNL40~Heftq_hHAWH%lV63dgINvF zPeYDtzMR41NRVR#t*mZzlG$@vbu8VnHP?40=(e{O4-fV*VonK90oye)y}B&#FqKN$X?L2`2vtq`(Yhg9roG&IV@7)#Fy5=(u zDH@KQ?&o^{Lx#p0Z7Xd#VqSZ(4VQnNir2BE0K%SRZIRxm4mH>V$4yMOo=?% zAX~I@xz_pr)Uki1axBORW0xJsoz7uLKFHT^v}^=AU-y~d4uj_CPS!DC>lFhmJQAOU zP5E{}yatk3IWIAa|A$qJrfEUK0PmhRlc}o|kF}g>yQx*Y+*A-&uuV#9kVIc%XaJbA|FykEY+gxTNfO%qFprNjT`DZvG% zFDU{q39n(DBO$&sE8X#wME7=FEFr1HJj;#HYK&Sz`3~(Kw(P9=n&%36%h$M@xo<70C{0y0|DZvjpke*}ulIEPC>(OCf%bp@_#fYbU-^IH z5}-8gW}~3|=fC}R)voOe<$rPk9N#N|5e*%2EAetF=09)z&#UNFqM#0E9O0n+=QIA} ztN-6V^8arhGvGR^sPm^hI(qa`!kiTi9}l*Xt~P3HhRdIA(YKvEMg5Hl`B6~O36B5D zd4+y7i5|LvYh)P3)Vp?1B{?Vfc07hj6J2h3_!CR2 zRLYZE?MZ+#sdi#Z`Q6YXFNWPVwp=$8IVL|(Pu)OQyMNYLTK|QK31+sVL*1Hu*Zanz~&`~8eWlnDPzWKjE;ga zjLK|srE~$ZEcE-Lf5{6!itAGhEEG}Zx}6gTSeQn;*_5Sm_ehBHzK4j#-DDbqas*Fy z-MaX-b zmdkWTudZA`^CD}x^@xOOCssAcSu8I93g5|<%0Zy;NaEfmx_{y@WFe>*lv?H}7^m0B zb2+^;)CeD#v7$yuVBoWbd~&vVGtl`-{Ym_7LfmnyolJYNanvMle`OzHdP0!La1uO6 zWu9}-wJ#;VID0|}vGigQw@%Fohn$ds>rwnteIuhLJUsot~wX2>U)R^_5>RP02uQycky1I^yrtKV!&!YsMH(`Pg*V+-Yk zP@tjF{u6`}PeedEUH;0j-=>Ji(@yk=mA+S_aljCrGjrhZo4 zOUVOX=*}$N&MPE6Jr!ZM?~N`o5md;gb(#w;{KQwV9<z$#8Z2fu)fyPDA4Ibp zA%G{LaAFh1V&NR4>jaXX*z8uEZj-yD=+s6PmO07#W9hPM?cAhusa|)Po0_pu%U3k& zXB^8)9{Bz6q>%CI##rC2Y2sz~Dt_)QZ*DJXbhlAP0)5GgM<}bH01;pNxbAi{Yhe<% z9FJNVBO~{T+lZ6XphQ~gnElou2!1^@k(}h6!a$oAlGKA4jnp1)CGJFK`to=A*H~ba zGm>n1du)4N^eLCIlJKseZSHRdX8&|)n!Br+iNBw_Yig?|iGPq-Mi}o#oNby#X_^&k zWI9dk%Q#HcSQe3P5q$bDg#!EgkeMH)yFgOtt&N*;Jt*PvD@B#C$6y0SOiQ_h{s zPFK8+!-4aqbMoBAo<$^w#ObuwD!=tmkMa`=6KOoKP}KfrJoxWobAbMJ-dd?^kSnS? zcc4^$Kz!$>t-{?KdJi==#I&fopB3_|u3Ki=<5=89*CEWADhljBC31nyyi>8Y8rR-3 z$6-r{Foq%YTIm&Ew7W{MC@l@Bl$mZ-r{r7M8GI8*e*QjESTx#JplTJ(1-|?LrMUv` zpxN(AAAFy)t~wWgWbK%hyW6F$=IgV#9A>$3Da)7E!t>sG#CcSpW9njL;zCowgf+MVJ|PL}%i<8Glm#;fAX zi~8dvrHIB4)We+02dtD?ng^XrUq*CZOo}@57|@-NMiev|tmQDzt2rE`C&cC!PIrgc zO@p^M*HXeP&&Q*E;w{u%<|3U}xe{H;{3TOUvoJKrv&*Np*z*1SPLvN#cK*%>qX$#G zOf@eE=B!l9?%&E@w(lr)p0M5xk>ND&l&&*>d8pA+jUS!YIh*|9?iy3{Bt_5j2b)KC z*JZ2mWw2 zSM{FlMpc-zR<0r*e#Uo-uo5Pt7haV)-ZwCc&#rRG=V5al;q2DPy=P(02J@8@e4g3X=4sygt$RJS;A9OKKOS0o$x~c-e@-a( z#J2I%5Q9`32r|o_wCUjlIh?DV!SNpF*^W}{#2(+Fc!cQyTBS2(u=o0UNCWgD? zo$?N@&rde8ZX95Wx^}-%eG?t7;17HSotl+h+15+}LB_jAkNDALX|=nKqJx7ygV(cd z?p02fWZQo|9C{t&a=~ZFYj$C__;E1>Plc$4?gvb!&_Bxg@N_c~VhJge@&qe|pH@{dUlQT_(AWh~KFHc~LZRwUj(-|y^9(kMl{$&bm zy6X#v&Guj7-l(P{9)93T*upT+5x zLf4`g^u{K_G#mkksGntb+DqJxUE1>H__I098jR!J5==?>hXSxf0+c0bH5AJ3D>|3U zS83!vRLoMpoL;5Iz+ByTHC~!WowZtf`EbY+hj6S|@IZ5V==9Lx5Zb|B&XLWhZC@<) z&yGh+Zo6Ts2Y2`P@6pZ{f83UAsm85a`UneKHiW$qLAczfEmw`lHJ$FlPB$_zcWv8wN%tDrhfKK7 zt-=J;y9#=KC#?kvnj!IUHi=>URNVoXpEKh1ra(=Jw%d(Icy%Fh;4AM8Q8@q$&Sg9xf<|9qt9Z#|BJ z#z>Zvn*8+Cq_V`yKq=KP-E@VFV`n@_==61f@iD_>o$JrOF;c)k1yG~%>=pyhQl4S^ z75Ij<-gL`{8AdS@8x|IpJ7@(F?B;%>Y4y^b0_gxi;djToK(=!~(uX>JgA(ATGj?k$$YXXQZBJj)RP1XeMP`}$BLUv1Djyg72e zqn-B%hOJ85R_S^2=b>Mbj*cwAB(vXIWI;|)&3hBQ?zkbug2ztPqA&C}$M(lnaCLk# z1>c|0B02B)ho`k%T7@tk?tlGJ6m!Qol!MYsLu>tTZz+t7MfF`~S}>+-4w^mB9N?-{ zRf|l{Z$>KP?=P6U5Jy#$7Q4p-_-ol8ZPP{nxATBBpYQJ1gjcfVPs^M=v7iJ2$nRfL zq`*LqJ!*g{F0*X*{d@UMToO$jV?HHow5QNzSz^0pqJY3k(EJ|EP2xOQZzm2nV7g^I z|Cme!H97Bc3Hixo6lQ`1kt}1=$yap^K>Hwav@30L&0>`A>$N=?aVZY`B%`bVSF;;( z+NzDhKTi|D>Uu-B?viEMWX;S-yF#+^(gm~iY(72_)V<5yCyyNBJ(#kH)H_#kPu!P` z7>j+kOAC+L7wxBOpD=nF%y(rq9IC2}LGYl3lC-qP+06Z9SlHB#3Z<00<;jJ5DjS>IT7BqEj~%<+EV<Fa}dgjLN0f5(D8-8m*0SEjZLRV+a5ko$ifW6<_TT*lblT4DoOhEM7I zROAak%4c5)Z)n9V;>O$AzLu$(F=JW$+wQxf&@Q@4jSznu41A`yXj97WopWP*!T7Z1`@ff z#1GC{A%NnCe8Wl|flxuuDxIVN0Fn5Q+V{5&0|7WkKRMU#^!OTn=5-v7fy}ytzc~wR zaE91EQlB&*bYz||Gl8#N8I9@CSgz6>tW3XkpXrT(&FwM&Fyx%!B;!&n)O;-DtjV~p z0>3BB*RBO4hwg?-NIsJ+fERkL{~s@u0BI?UGBw2Bi8X{5dWw7@gF=}aaxKT)={t`i zdi{7dmA^GNvrDx*P;uZE?l|)9>MMgPHS0fTMp0cS)acO$!G6bD4q>0wv+ zNSrOT;$sq6uz-^bh5#9Uyx0`k^Zu3srsl&-D#s!U~2jmF!_B4aG5b>Uxi(MBuqkH zjTqV&0iGn1RYxL1vDY;~@4$j&U3@lH=SBU2+Nd1VZUS&St#<4Be@oA}MSLX?oDOx*g!Y z-+=DwN$nfbRlve|pIvwFee>px9+cc}I`7+#)+jo1Vspi!_~}4}Nkr81d-FBx zEhayFPX`F))biKVEGp{i>RY9`?7*#Xw$v{N)BBnD=VwdLIN=xHcHYM8p_JbHV^rJSd6?T|LIiJ8zBt3uPZ{P#u@wXLDgA^ zhG!m4MW5J1<%cvfNQo=zy{etu&Im`~<{K9<%@>TfcjwwV4zpbj(j5UU`MjRiqXRG~ z9jP$=#}!jTLgEzPDz)n??CG~vLhfg<-wy#AW@6~*(57A&k9cEiC)!~#RpRH+3y;q- zTQDLaaODwBJR)MKAC*lMPRIi7wzHi5%o88451zc!I zKSaiW&!qc$E4?fDgZ8Z-^>|cQVIDZv`(_{qoPa4*p&-?&OiWCI(L!`+xj55HLr7er zz3EVmWcG_p(T!-EK|9xPzEv?C&*CEOCg0x#>cI0n_LlKe;XB%KpnvAR|A%=ID?~IV z7s(F=@BPZ)X=AtsN%N`3F1zBM1KGN-R+Mi;(TLW5}-K` za!}h5j}YQ98RLug+5-kdZj-`wF<~B@_UcTPd=V+GnM=B+ouEh59z1xk@6k<9pr{Sd zv^p_Os&}3ElrTh;XmajV5SEAEhnK*K!al9t{8jfgp|Bfjzyy70nKt-^%HY;h3sAPK}yRe!1yHS ze)SWWam6kyEF67aiWZc&T1g%9{M+TQR~dT%@O(nqF1L_J+wu-^XrmylPz8wLvD-Y7 zYj6#s)xa7LJi%ywx|6Fwg8+{Jl2&;=S8t#AJe8PZ9qKPo(Q7ua+_>*7%;!O-Y(s>W(#9g@B5F&+%}Z0 zwB$Wu$dT`^mxbVYkuK7B!g&Q(W)4f;s#D>46Xz5<>2}R2{NU=tjckFLOEd&mb6}#b zF4a)Y?IQ;sK^K_aeCR(PPtqdCv*%b9s|21?Fy}L_6ORv)NdZAXa^3s#6Sod%MZ0Ct z6x`O<7RzNl-9mfS_f-oTAgn{c+19M{qhbb2muk|T^O55jfYN1^mCvol0VAuD6z`56 z=X#5f5H-UM2XuG*i>yjoPK6I+czIIVCMG7HLw|$!VO7q_d^hqXs}PbrwX+TTMzVjz zqH*LcE{|LQdFjRq^n+Qznn`yx3jLR7M|3&%><*(}d$(947favu$=SkF^Sa5)KI|>G>o4 z&K&36qx4Ta@?Fd7yuX|eNDw9=itKS^olm-y^&F@N9t{8&{|1z3)Olu|ot<^zS-WS1CcEB|ijF#5X$96o}=%Qs#cv*RDOc}j5l0@CvlymrzzL#4yZLQev(go5# zy(tI2kA4wzq2O;Th2naK0E=B`4Z3BJ%?v-&M}WT$87rUejX`!veUmn{F=Va2z#63?iSo`%v?gExs6z0DPwSvuDq=iI!OC zV*3_zzfu!tcU_PlMT)tIA0;D%ytrHb#p*BRm zb-nra2R$`PhK8xNMmoNu)(gD~1v^M<0~wLKNvJ@Xw&MdaB4${j0DKpL1#`PXgpeGOQh$7 z^zZBomLGdiT>*QbRo|LG%XX!e1P#gA46JS)%-+#&w*6i9<11hk_(t2#-}Ef4z?PK2t>o(%~JIWO=TSaejYz_^rI4!#o_b~)P9>K@)-hxzBA zm_-C2yfODdsfOEJ2Dgavbu!6`1w*QwBtp>Y;YqPHfBT$&6n%X+0qO3`D=1(*DFgbr zol$Q#w>EJxe|{^gLZkwWJGBP+C5L*!@u{f>m`QAqu~wn?jfit1SWYln^dHI_1=erp z>)RQW6qmY&|*X(@ip`02?K?Z0V#p*G#1g)6Q5syTLYj)NqLGlaTw~XVg0sazHz5M z0-JWAH1EzA#QVFZ;g(B-JeS&#!dY8e3k13;0bj)*Mk1&sccGE01NU(EMVB$V7W91~ zF?PE_f6Ead<&q#5MUG$eVXfE6H$@;e(4lGOGW1e}0qj-F@7^Lu>F66~K}8*)>3xoL z{Ii`mn0E%2oEZRoi)TmefF$1OM9eGGgwRA$i1TptkI*@ZjANmo zeH@5$Q<&oqm)>cHZSJo%XG@Gpj$&v+r;u*2wRRH#!$SeT%wb}|v8?`QM9=T!8W%^$ z$jqD`lfpZ5J5&ivhxXWSq7eemz~Jd;Sr()axYl7H`MV*$db*!*&VRO50a}E-SBzMH zrE6$5(AX-L-EPdXcSJ$2kI{QxC;Jqkg!@->keU?`ysy-V9X`FnM#f^|AOYjh3K)V6 zs6Rz={;6*%pu%o;^^?Kz?Mz4j#Asq-HS2?19~x+-3DY{-?Gk@m0;e)4knR1iB@8&@ z3Iv}NbB*rb?=dil*7;w&I7?tih&7E}ynb~SnRmb-SEm`dGTbIG1WBTLK&fIKiVTO( z%7I&UU%F>tx%O(Fd@=gg-?C0(7{DM}Z})5a^}=hk4p+(T77DwFR9Sd-ZJ^-_yD{ho z(!?-hsoPD2e9PPx|?x{3VF0LL&jg6T%MF7mRmkT*PW2T*v-}@HGpF zRTyq#?Su6qLh)0MAtvOf2|~BuoM!}_DNb4&kWWnn2%W`#!0=HWjg*x1dAlV0dDI8~ zUv){OFr7C%<#}*QMv5G^%plPa25D}0kL?T?6|?qkdP<8lp2%cK$hgJX{a62D0ZDgS#m$D!zTF9bD4WQuQV zwO8P!i^rQOk2^j5Q^RK9GA`M*06_bC4VZ;Nymwn~St)EGP=fVFQv@(Tr+FKB?Xjrj zUAnrRj;jgj?Wt&oEYOrHanvM4f49js;h_04d(46xu5V zLK@qRceevsR5f0w1~zsS4Ut^4FxQ{-U4l)mvOGARy`3!aJ-&qUE# zg8Y9tb2I_hAd-ZLMhp2ms0~<`bV#npyRfwPlaL@E04n|vOF}9f%T5t2Z68`u1Jcb}?M*ychJBLzI3<3&C3mskSCiNZ#K z#|ct!(TQ&0ylbrvHL=GzhNKWA)<82Kn?1VmXHzies~`+_A2w|__l~wp^Wmv9n+=qG z%TUSp(dA0v1zy3=)8J2^vExPBW8;Ha4=Rc(<)f|&Q;}~cZdq@Q8Zle5ok!)m`^!~C zdfniE@joQ6Uta?5Tb;5{Z}`6JL&A5r4E%sQ0OnVEkRW^?Hs*i$@yv>jd0k@j)kIf}w zjy)R6&&arTRW3>q{O?~;CfKz1kniUB>kxdogggWi*V>h438g{zde21vf#E`gpd*!C z>3m*;9^@T2-W)XDTdEglh>~Yxk>i`jq$>CbQIdjEZM)PxmZT$(lm-O*A37dDYiKpt z@t5e2cNiD z@5C3Egk}zIYUzO;Mdvke(Zm7SK8xXE<%I03e$V4j6p}8|bMkZ}9b_?ZkWb&aq9A~D zkUgdUxD)S-ED_`fInn&^A`{T5b|;jN1KQ-C)cKsJWtLG;xC|zbhzDa8y8ZOPbER$0 zm(_XlBr!v+m{sp>cG)b`4rG{wsOkS)+&{P4RUXu=RN~#OV~_lLCI9Vs#Q6i`-N^*R z@vDiGv>Z|GL%uUHA>q=vhFR$ZnM$R7;WHCRQi%}QLP8|&Ws2ee#6f201wtX!MMa8L z+oRr(Jj70_P~?u%BqF_WSu?X^FCFp-VqIwZ3tG$tSDqXP31S|}d}&92^`srlj`o4Gk~b#E~%u zfmk+*F-me*%r7*?J0Zv)bN0H*9{WY_#Q6PXDL{vi80ddK@fG29D z&l3sfGgDFmAP2Nncf0!o+AE|x`W$Tpt!8IuC$9R`FXwua2wcIe&Dh(=t2;wzku+7* z{|5qL4QIOx%^3{KC$hjom(Oh$$mfzr`r9ywAT736I^wP`ibcQ!eTL~_Lx}eT6`w;^ zI|={LdA(KiIizI1froGvMy`ZG3)tNhR{JkY;AUOg#s6VHNpSaAl282g<5CSpjlL1y z_v7S~Hic{5Pvm~I-zLU57QX8r_`!yZLk%v>=pU=zX}f56vXCa~l+{+EguP3n54AH*9|OT-%@0hLi(%sWw(PzzdT6 zsK>wkxH^0hgEGy;=MXPZ5Lr2dBff#bS~g9mQsr2ROzqaW!aNWUDm-ThmmdJL`Z#Df zvY2KizIYNx)pG01GgMsiX4MDQ^viSA6HrL>VT?YA>G`F%;9|#D1T+i=oN2|Um%ksE zGy}+7A0b6OA~b0@>G&IDq%Cq+t3h~IKVn|2hvsF$pMug<&R{UU1An-W_8hRZ3P~|g zua2T$CL<%GkmL3Ez6jf0%njAe4`=@weIgKy;4wwysaz#NJ^|u9FrcDAJ>k5n`>)iz zvLt_Yym5Yy+nY?$J!D^UT3Xty-^?A`y^T5Y+Z4r%uP#+Uh0B*4!bjSM6!T&jh;ykw z^vt{QO6Un`{XbDmTqG!5t`X={PIJB4{?-p?L8M#@Us$fUA%4N5%Z z=#c6=L+$(nWc+nW1}R^%s;U&09)IZXM_kxgPhFAo4xj`{Nu>M5lkld^Vt>Hbzz|yq z!5WZ>p0asJC>x-=OY>}Wl-(ZuBj!MqVyquIgocD*kjf^{sPO&r4hZT`!@~EpeD63` zJ~XVZNY`z3cP_gD+|R2>mCek?daU2To;aaF$OXw%@*W^8CGa{Nn}>A3IMQNqkx2_m zO8WkY=21(6-0@it`m4-5M0Ue9u630z8|hl#kaKP!`ad!8@Lr=q~(1e8JO<%#{7%P4E2L$Q7B zEG zMTzpjeynD{&D#;(&)?KW_o~%5IfrH(D+)*@8cd4^^fU0)Wo3z;O;Fnoqn7T@rr-DH`Wewa6TGuF z)c~#gbWJb88%7!gej0n!_@%09w(Oy$xzBTl6^ngy2WLq$GJSbfKv9xqy0D!<^g z9eLVmcmJ%&#UcIt_a6#^e}z2_NW7EEOF0d=PR$Rr!QPuIq1g(qgTAu@gT8fxYPsf1 zFEF!*bSKL$Rhd|mqMI>q?n~*gJz^`gIk$v=o%fa-$I9sVFO{R}deM1XfVF)7X6cyU zJz?qAb4KmP52`nQs+LjL6OV{L+ow*7M!=otzM73i~fE^q0PI$X=Z@5e_qAj zV)`dwH{z_PJl<1!WIU>uFUyGB z7lu_bo&&_e zyD5HT6IpQ7v>m~CR;rGNzjEG{%hdu^yV9>Px+oHPt^s;d8J(59ey`I*>%;D9+Qsb6 z!8dqmX;lw1vQlUDe8{rq8n(zpTR$3!Up>^SmG>2XjI@Wi0FEx=r+PPILGF&y45wAM;%XR_E7giyexLz3D6 zD*>rx1Ckzw)`OA=={&b$MMBnd6~-jIs?dWBPu=cETKq+6N65}kz=s+4?qhmylzk?? z@-RhW3=eGtB;O;5mI}O5@tUd30rL5Q!j|c;=hwhiBrBY)!`AO!N#291QrrSJNXlkyl9J}=_y(T-`+W!ttv{mAIT5H!V zw%JD7&I+l5L&S~F;C#K<5>-4RQSYkRR2|WusgEARI-sDwz_qQ^gq-YaKrgrAhhvyQ z>F_O7nkiLFo^G)cO)g;Y6lCvmONdrc;}-Vl450AbJ@;$JS$ak++vQx6RoOTrfNn&^VObY?bD}+Nc zyKMdh51@Jxm&32M>g1BQO`0U3ef<2Ta}iy)>}fuo^V z`P^~b2yVh-4{Nv#!}Nlo#B<)G%zA+--=H~EPKq;eB-Qbk??I!BQPJ6#sK2I`#&$$T z?{upuhqE9BP5++d@V_nWU(nUdX?&}@v-=2Cf>>k?a)fO`!R@P`Pk38jm}5oJj&_<| z{5T?za0N{UCoa!m~r8gGnr=FFa#e72J+ zcVc>mtjYKHr%@dQ$iLmQhzzIOwI#)}PWXV{f1ID8QGl$B+~r0m^W29-ejYC9k$#6O?-b|InKRd5+T`H6G;Dgr#5t-wA z+`K=G^z4!C@7E>K6PKRg zxFYXX9cuP%on%n0RGg{m{go$Ll7#H>b?xm~$B~{`9{5c`XC5NK35fR?Mn%pvAHE0fkQ9xh|rt$t%j1QD)VZ?wN?0#Qq|0N0cRD8JdM<}{aG6KRC0A*laP~RYN)EHFmiEZbP@5+=X`ea3;GofA$rgq zdHus@cT@^3+uP}Qi+kRP+FfJRR+8ZM4XvOiuGKftXOp-~`x<#N1i>JkbrmE6?;&6Y zjaRx*KBdNdA+#ml@Mh&<2U{t0k-cD?c8V%d#TQ@ntp|GnE3b<|c1@WEsx={ZjGd#K zo&&MtYTX5MVfCri)#s%n?ia@2>H56|fFSd@RhOniWRRd{z#?hE z#~~phLBDc-$twIXIv07;Qi+xoO@mgSxDYlElBSkj-ugy|>M-Ie+dV~wY_W<{l~X|8 z`~}iAFzb`Oh5RJ}QEcyV<9=}Rd^sfkg=ub}-5r}5;V$Gw1`2u}fB6Av^FQ9-N`$oE zag|wW7zvbQ&MFb?UmTW^L{=%FsPwA*cG@}=D82#ZQtjAEhej883U&~EKY@Csm0^Vd zjXDYkAD(@}A!%%H51h@_gCR=$nRRl*H`k5++U_lQRQ zAP6WV05f)hdh|sovX`yRJ-HX_omi~(lpyJTrGUJ_pm1W`)&GaEw~os4>$-&jQ9z^w z=}rj=k?scR4iQkgySrOLq)WO(q*FR1rMtVkIs4`p@AH1=`NrW7#{lmE*WTCOE9RPO zE*x4NTX8^rhBRL>SA{4+diFp%=gO)S&H0#F=6PgjY*d>aJ)qsL^+hUpZ=W%UkJW7` zxXAvE^O$=BxI7L*hVU$A*zeDQL$n z8fxwLhinO*t8@JP#mpa%`n9tiHdK+246*YR}NKzvXBj6 zk3I!E2=O{z{G70E`!@M%*H~yMWS)PP(#9^{`Lz5IW+3bQ3%imAZfV?%?L7p26{e6o z`Qr)V8c+F;Z9VYhGA&f!Xb&*c*zVCj1k<8(K#nbZ)(3vsGj0&FM0QvKeOORFfB)&z zcR(0-Q{uUeoYyX99;Ev!rnKJz)&%0-)s$LtIGK1uV^cmq0yo0M zi}Upgy-msZzY>yt_-&rzzqyf+e{hXb*Rf>vjyq-Wabw`gh0|TyBu!4&iu_hZDB;`5B4^Gtdk&bWQ|W3jA8i~GIEa6| z2Hrq6S?T$Zu@rz&O_W;JqcPhSFq6U&#BuGHi6K_=JIq1jMJiKtbwdMpUI3`Ov5MY2 zBVZJ0^+n-*4R|KTauUyw@m~W;%AX)|XWqm{`0EkcY2}*C5;uqA%)klq3atGz2Qy0q zIrY+B%`Khloyy>2Uiz@7KjqEf;{{t(X?YzFH~AG~!UZ2gaPp6fSk0IIV4N)XsyhS; zcLs!IK|)ExWe83PEG?3{AEc{8OH1qDA|z@ll>3#_(^_767~cDmnAh)*Aud?~LdMRX zorPR~@kX!y0U}Dp(gA`tghTfif5noKS1_Og84Z#@g?`vihy_JjzSWQHq%D~mkPPYr zArz8O>lyQE37}*4FAnYP^wjKp)~RCA1@vX;%+hR0IOei>WXgEQJZDd| zm)yFm!2ub`?iE4D`eFq3lAAb${rH>m5R^D1z#Vz1QS?=YQ=+71x%+CR2nvcHN#`&%2!!~i*R22XcL+K~L)YUA2wv{*m?p{>eRO^hgj6I!UNJrkO|a)0b8oH|23om0QH+! zMNM-M;nUr_tB6>6{T5&h#9PK(jC^cJ@xZoKwt&%MWS|x--D`ekG_-JkUJXcF(b0Sf zo%DR;Y)8A>Cs9G90&ysg%@t9e7veQRq?lVZ6e#{!!|TTI4pFYi zTJMT`o6R}e76QW>h=Jsxfd4e5NX?};3`jK00u4?Hygernp7hOFHm96*8a#0Vhh8{h z{0$9FG{o6dTSWE0&hTwg{b>J92@t~pnT))8)<&|sz#R++UOsaXftxS-u~kxm{Bj@6 z{{{@A#K;b?g4`XLb;A_<*jXSPInv~Q6S)x}>2`m$p%2QF#~_&sAvFFS>jr)24*2c$ zfL}}TosA2y)AWF@;<6R{%S46*5-mu{dIE0l&RwhfSfJ|9i%ONtc*f=&E)Mv-m+E)_i&Be0JFBLka|yTKE?FJk98BMbwVnx1L-w2lkxYeGr~W%OzjCdOCgiY0b7UA zA21wmSKnl%K+Ni%0uDzi|fkEt`1}9LZrEvp~`er*~yM}F-1CX>6VEo-5 zXi8Ycs7^3fA8Un?u_C88OZR_i*Y?Y7#dF%J|3!WBnvs zq--J|F>Vni9&vvMND~v2)cBAJQX+;`6}!}N-N&I-=VUMbx3D&xDKdv_IAxII3!fzp zGIDC4bykQW#DOhfYoZ6Cbo}?%F(i@@Tai8GrT%%4dLSBCQ5yEGD&TlRid#(wp)-52 zyzbZCz2DAqlSVV0&tB{B;MOeVh`oUHbwJSWfWWXp7uYO9goK_Mkh1B4_?4tE&v%70 zerA+&!ll^!PEL4nyg->&z0%}cOiTBeP*%(caUuB&ok8 zlT6}!Gv-i*OX_l3+CT92nXxgH1%%i|&hPbI2Z5g)5!^VCT-Bz?e7zMaRq0sUSfDXC zShJZ%M2RuDdk{FB=nWU2LW4&YKYgtu`tWQqadQts&BM4I>o|#62mMq?G{qpM1N!6_u6AK^m%JQjHNnNh{75ne%m?1^^o#Jw49t8wLg}9+*n7V2lN;HYB94SIOkw z!jSTWWi)P(8Cv&eZ}1>d$zO8y%b}EIv%L!Y8DHI|h#sXbNy| zbPT*_zhq_|1~rHFtLv?QR$vW9WdDHjjg;q@t@>Msso)VHCCKT3&<4^P(2^Cu07R_% znY8=-d(iTVief%k)KZwOLk8vnOH8+V1fT>w?|~LyNFNEEW{n)BbRtH4a@0Wt$2vAd z5LOeA)|c%#ceWp^5g-!Z;e~5bud~ZHvTdA$@PBegkSlpWOGbcho^BR}3Natwf>E23 z?bu$vSpQ=!KNzTP>?&AorjZ{kQfFk?1oy_ep~AoH-s6;cetj$A-mfXpCH|OsmBj+? z&xgXs*KZ(xoDmxIVdtw^2ZD_Jy;`cCpI4sDXODUVX{_G?f~RMAenbxCeR{o zS}69=bjSJ!TXg$d!O1@8-wX25{wt3dB}x}Ei_^3T-t@{w_S@F6QT#&~*xh;z8OHqb zuB;vKs`aOS=x2xtu+0Lh92unX!p8^Z>QCCe-YwVKnG0U3%7waY<6RF`+Tp@DXVhxr zu)q~44}jwEhtGAvgm?8Ih=gfj+OkFi5$aoD z6Z6lffKjK341(0cC)~7C=jt8t1WEzlkVT!Hr}Yf*q`yMwNhu%Xtzy)c5(^6YnkBNIUytsCLLP@;45C`sX0dg5f;^5XY;ny73v!Snr2V6SuRXJp1>!d zQOxQmx8^ZQBHh4xV`~cOd?Js@P>Y$aoz>)5_Q#w==RI_whU<=jVr-b;V@3g@Q-C`v z)Y=3D@rdESK}O`TPw$VSDj64oz-VZWwNAaBau;9rBcc(g=@Qi5x1>(Y!{1`!UwbZ_ zjq{jI@qGJ`<aUi<;JTwXtMr%2|O?X zUYVPynI{`2t__pBU&?6isXJ+#lZu?Ef<3`Xif>!aV@HJWXJEsm~p=eNSmSTM4;{OTpJs%b# zWPzh@=HH4_Y_D*vjgprMR3tx0mmI@r6bU!jov+Z8M!d@ywYUk&|Bm;qx6_x^x(c`G z`q(fOi2r0lbz-cT5-Kia$BjDkvOI5Wd`EQcmN{XC3du)W+(16kKS3D-AW&5CJboRh z$?nN@rA$W8!klQ_iB)l_w0c34s)o#*01qLcxIpz&Q_p}2n#E^!3^bbUIca463BEC1 zo39rP=Y9xu0(B`W9M``BB7k;(R)TD69Cgcwxx*ugRfX^$*g5?hhSTnHTpFifRq2wAZG zOgV)r@rZm&*uAs`S0oqA6wMGL)UIP07UAowIjWGt{^9$qGS{?0+Wi#m0=W7G51Crh*v3ha>%>IC3H!n(|FubmcH4lbXEERj?Ti(stFF33|RoeJ4{e&c{%>ZM@Vu9zh9(k48mBqJ!gYBnr)ZJ>k}0 z2YTclDfrYkD|1>NSNa?$znNG1DQQdiR>GKtBVfK5?tlQcNV8S}j4HTZDstLyt0avI z17QWj0tPsf0sJ*kWe-|k4B#a9y*CP)X38kVV0TR2Qfcz%?wj4+uLuygW4Dery5^T* zwtsmT_VD+}yF1@m@sp!i?L5HQ9b8HQ!%mgTBiRN6FiFX_$7BDWatT4&>G|MKAlKQZ zi#^$};}pWb+Pmus-?uju;3Lo|ss#s?6wQbRYIm?Lbwa>~ zGc7;oibuH^4S&j4fY-9W39eq5PYVvJ@mtdERC#rz*PH6_;lPPIr@7o}j$=}qMy(aq zi07sEN3TE$+WZ(iob<~_H89{EU|Mp_FrWKFf=U_U5vapK$DjFM+Ms|j4RJ9!#UAtS zzPpk`KWJd759d#JogmefrTT$TW=Bv}-<~ge#Tn}PUF)s`ME;2{d(o_oxnJ=WY_%+? z5Omhcmr{v9v41v@5pH=4^Y|`HuM!nYNnB^GyC{C@Kq%!9X#x|B=r`Z54Bi`nTs#^i znvh83r2<7Mkp(o;P<#4#56#wfY=*G9?t8@zImT{A>lURr_&7} zY6OQLyjz_6xvj~gVf&``f({Fm>g_+>N{R|QI1U^mPM&b>?erCTG*pOa5AX1a-K3{j z^i3BNGqiH zYLgwXPtR{%92n%C+ZUF=UsP1ILW9Y{GBSI-GZSv54aXt3>uk*u?|<3Y+J?8-048K~ zbh9#Z3Y%qWvQ;aB1ob@xZG6*Q#7T?dk*W39y zr}QDzs&^N$iN!?u_qSqb(nqtS(x50w(QXa#oXHT+IYl4y9O60S`7?mmd%oJu=F?IV z?QJqQ>21=#csYJ^qGBPRBK8V<{MV<-`!H2$rlQ=4^IFNBxmXkr==$f|Rf+`dGM@(z z8k)c6!8G{;E&1OUX&7f(aGPgF<`)ADSt^ZoxxB-8fXf>iO$4)!2G0Ts5!a20JSDgZ zBTj*Ize60diYuz*say{QQ-VH=1pqZ-|CX6c7&O@=NjeP&ER)7x&_EH;tn8y^>=hnch?=D+M(G(J8|FL2RtH;ydX^ z!A>fCP%EY?ygvSd>He9} zdYVytPJ6;zr?IMRzi;P)vp@Hbsgkx9k6Ut7Qh)ySVKMmTY(?2q>tgQg-9Q~f>CI`K zC($0p!1v@Zv?M3R_o?X9lnbYAhk4kNQ9ZQ{ww?1=Nv44t9Ao&V)ej;i%I~IGsP|G!}aiVzP z;Z{K?1kWI&n>Y2Q7C8R<_i-sS(@@&9H+OHu!|E9<7D1XiP&_H1U8s6VXeWXXJeybW!k+Lz_D z6}ZIXs#@T6z$U7#C3}?o9vxiu=2vcztg#4EG@*GT0wGfZ!o~<16f#YL%X5bXPF5Q1 zSxq>42z9XC0xlSA5&Z6 zq(_6OQgVDMS(U}k#7pI=lE`Em2Ne&-`bxGpyOp~Xxj5gKF6>Z0{DQySwojbh&gPoj znumwVt$$j>2h;Nk{t--C)#!@80ch}QL1D>Dg5PHmi)V07z^0(23{2({g=I@={;A+fO6?GJBwickf*1w7eOE3$9a#NJ&c8f85Ro2Fgs%~ zGih@xc?vjYKt~7N(q>W&?x0luUgpcS5bK8#SOEPCN5mSQT8;$u0NsLf#CTFD`~nuG zL#2y3f;t9mr}W9OKKUW_vo$VU?>};Jkt*8!aK#`jQ!Z02{=R$AR|r?rxVEFrIOFz5 z7agi(!Q<}()Uq zSww!gykx^;%w#!p%9T9gp~pP&3^iRAcN_f@@v=;)Uq0{ zuAQ$TCJ{>GPFbCMELs*3vUYX%nVjW9rfg-6BC6_5-$$pvrJow3m*u|>`FPj%-N4jt zdT-fgn!L+6?iN1~W0kW#A0=LA#ql?6I6oRHW4@QNN0(HuSUDZtV&N1C;oWp2x!#*F zk9vZNjpq>>ct#5@jV_JajWUrQTTnl)O@GQ3KcNFMFqNtd zX%7`1brTg-+MM0&b8BzV3 zC4CzH9d$Yp)KQ+N)2OF~>8YpV*ocRB5;APuJ6nlR!8o+n$lS5J7q?$1KE&UG(bmw+ z4`1{H7%R&PyhSb~jFgPfRafLVcG0U#DT;PR3 z>^=ndyfzm{di)OZ&!6X;!#14a73P+{ByKrxn4W?&UI;&0V7$U+RvI$`Q*B4rgI+(^ z3}g7&wB_KyuYw5O#>VFSD>(u&SsG1z1&hD6SoWA9 zE03h}o_kfPuy-y#!TJl^MeV#CkQ>PO8%cV{77$b+?mVd^8tBCO?45} zYAI07=GHS;;fc$%(;KGy6Pw-ez^!-#QoOGp^^Zz9w0<3^X-r2Xk$QPyqYP0jJHC+o z>n(xz;=PLrEdSh|>>e6_OBM{QkMuMn2B=0P0`Z3&i+SZSNQnrN0ZZ!i?QhrQ_>4E} zCS4Lps+>duT`pl_)Pn;x*y=@qL_A{q;1|AtsMIbYNu7ZlU>f|w;yI5PLT@vS*M&Br zsZEiP2v1zaCIkJhM<&QX+>V zu#5Z&!q&$YKw=wcPhuy)E3yFrYad7(AtJ|*MK3uWF=O-R2c5-`JsE=Gk~GbdKA9bY zOA=pmEJ7eg2Kd^0*k$mz6412(eEz37Bme1JdZ^oJ)6$*;&j(jfUyww-MH9m(fy@9z z3p*vEV_=8_H5XxsMi^DHl@H%k^bLrQH6fifkn#%Xcc*$|Ujs-YYt0XLmffyk%$w!_ z?~U2j%)&jUDx(u%4jzv^7#`5IWUACLx*ZCBXUCv@vk$&3;t+2qJLdlRdQXkq5mJA9 zBF$y2Qjds1v%x*c+N1Y2!MP&I{5;8aVC&B*ka|)8%}xEOp@<3mRunCQR#uHaxalcT zx#8Wgv>Xro;281Hbzr0xEzvKJ_X2}N55w_W1Ir6(-H0j>u~ZFUq==yqZg<`3pz7PD znV~6LVB?`6pZ7w5C4u=x1N*K)7F3)h7MeYC0R>_R%+^{1c#g7Ct_%SL?*?;wg6Eq( zOFw2zlxi!2cd74Q%otQcE*|Oi!=V2?8ky+r4p}_AH7P_GrU{clOA6r8IiOHD0>A-k zkr0fZV2)a87~8!%N47d?K=CaJCDkjBEDYUkhM-Tn#@eDQXe77Z$yyI1!&m z78q3e{=4eOKv05ssRaf)MBWhdf@D&D51I85iqZ1^g__C=gbXP^s+Z|Nvlx%~;j&ps zsg`Jxh>3}P<@Y-rP`^7NI|ok(>#?4C%G?DTdt}20 z`GE)QWh%yP33Wse!e0*r*jxP+e8d6Ft$<<$au8T zA0Hm3ul2=!4Gl$H8%X&OARoXFncG=QG~Z!3kosBIt$Q7W(Jy%ZRK80|xEgV*MAy7?*c2{x$_ z=xK!v0#|^auQZKwpxXvyf{$O&b{JQMIxk6zT1TBD6|X>_p+bv;qhs6+-5s?;RuD!h z$%w|wWfCRB?nnWELys5#$Op1%a^D60H$g$!hU(ljqyjotXMSd`5Ck`W!Rw<9T z@6_`P0T*SPB%jNvcopDYBYkC!OmYG}{nu?pB|>@CVo;s-m10oq=f%r$=oEsu4=_)k zcL!sQ_e?ga9-!SWtt(yIIr{Ix*-BHQdch_im*fngxl!)HE*jTtH4C@=*&(pbYi)*2 zlkEfd3&FeWlc0P7th`ij@vB4k-?#+xU9j;jX_Tm}KWic=ejf0i=iYR0AhN?)%2Z}q zmX^ZKBGl+p>X3O4ArGv^80Lpp*5dwc#V~;tgR=q!eT&k5)qSg2wFn*Zw5F@mm;MzXegT{3mBJN<4=z(}9p^zBMg{Ev!v_xxP z25JZe|j)h@1ip|C~*1O3e1YY8?FwBB%vlnu{naiUs&rr(47ct zjV`#iIh>g-wjYE>_F|f`nhA6nQe(EE}`OThT{oBfR%0(wwhi1%M2VuAm)TVxJ2m)i>Vl4x2!O@Sk;67`Uh}T(#cSZV0E0n zk+xg!O;=`D(+H1%-b;DF&e6x#bMm4oRbUiPd`JBY5&PK^b_SE+;5K9n0Lt60=Z6~* zcmNJ(wiu1gRBIvDr!0IwK6+s%#={~ZPX*Nak6!er_{6VQJ43_Iz0Gm@;KUzh0SHs}WGanD0p!Y9cE zK2H|4lu1jI3?H%Wtc_JrW5s(DnBBJ+Y$eKIFrpqoP(Nf;Ug`v@?^Ha$IGNXLIIPmS z+(Gt2T8s_4-Y z{o{JQ!}-g4p8 z$|9`LNL{MFMu22#6qE-TbAMF7?x6akSzJ5f3_LmbOfe#8H17Ijh5OJWARdK5JDx+u zCh&mB2?Gyt8yl~o46OifT^p}S`p5?X77^ys%^t8`=Y0ivk_SoBeLIxb1Q@+rWEZ&} z1aU4IUS32Z_%XsPbX168Ki5CWyW%xqrYY$L$Bl?o=fHgz#XeNF?LW^IYBvHb7qr@u zgPmOrVfG=7KS_}lXg4zkgRtMuo&cw)tXwGhO(`HWI+L1-_0<&VN~WN z@8_1PP9+*|b-!b_Qohh?=*&6CiNmwpv3-emp!nSn)dMG%-THIw=!1a844~!ignAeg zEMZsJQ5Bpf?tP{*bb8w@-q3;A*<_|zxFmMo5cY?@u7fjVtCru z*X#Qk5@7uupLn6ev%)Y6!a^qd1sHdZOfGhC0VXO=E6&oFC^A-nM|F-#RfTsbQnbsY za;qbXn7PS8Tq7E{H;BY2Y==laErC!M{P$#Qt)L{&UNf|iZ*o~EQN;%fKqgW=TvGd< zc?ZHk2&9mVm7*uPU+;^P4al?@N{41K85OsIDFh6!p*;D_TOK6XH87}3JVM)V1u|#e z>4G_P-4<3MaJF)>2fJzk_; ztvOh#M}Sx2p~+4;Fe2YDCUimkDT`Ki0nGMw#|8-B{#iVTk7i+g!l7O;41#w2KY$sN zjcSI>)EIPBxkS6LBgS zm#yHbmae8+8yP{wNAIuXo=dOQAl@tq6(ENwfH{!?aJ}PqfKi$af=zjd@M4qRT`dXn z%xCzWaH~Pohpq@BORJz_^G%c=`ZiuoO^rKXg`XiSywk$;ID6-mg23qAEaByue4BK7 zH47eJjkLw5gyOB{0C>%EBrO-^={DIlY5%z{P1qM>{Vii`^)9|cJs?iXMZ}_4JH_^o zblx7Neji2ZkKtntvqo?OW|Srk097KEtwx{(CHM8V9>^QVeaZy`B&lcWY&X9ACNkUQ z;4&SjtL0-{%PQ8aeapeYF$^XvZUOcxF5)USV1edV-OZWk%#zg*(<(gnGbzU?Ax zoBc3eD}U*SY8lKe^@MtZ*l8ya$ARB4l7)2V3>F0|0)y(olmIh~!fC&K`r*V5PK0pQ z@E=12RufO56KKv6Vs3+9NHwJyN;UF)N z!ecdyT#AeOa}E@U>{h$c5{NLryMQ^Hc^{$u0z{_(c{m%6ruCgr@=rY@l+O@Blk7#! zW)F9q=Rn7M?)Vl&hhI4YtldD*J;Ho3lCYZYPXq=HCu+;7AO4YTMN`*33r2dsu>CvLR%_bn&Xb;$@{JkrZH%vCR*(Q+YP{*>9jkG55Gv!Goyj3t0o7aXP>`!F*YR>N!aPhVT zaUjALgF!1<4T`m85Kq7l3b9#WRnVFZIKjs-{C_s&KW@z9|J>@sd>4){)WMN@rqv7| z9+KPw^loy=C8A-24&e^e7#J9+;oEe_lX=`OBUAZ2Wk8IU3F0Xe<=MG8YWWOdzr!)_ zeh@9wwds-tG96P0x%pt%gC~aCDX24$&lDXKv|G`epP%>RZbHqyV_wB>vfqh4)pD@5 z9xfomcxS`V<_j7UU`1i#@qNUfcAj(jEugUFj^x&Re0>`>_-h8(v6oi8^@)OFK}S@w zJsa0Tu!jt5Yp#~}LsOUU9#=C1Pj2f#9(ef4tl;+rQxc;!1?1*1Lc4;SQ)B8@2Ss_f zt|WPVe*GRQ%|FQy<{lk>Z&6CdIrdsR%Jgrakp7snp6QW^gpjvkf=RS0+P5AE;_9UG5=eg{eSTpoJcv%!hHh6_YZp1I@ zb+oTJ`VW}I)=*V#qqBLcY2)6F2$2CMGNYM>*1z31aSbYb6^xZtEx)G;ez*Vb`no8H zsiT}-3YGFnWg8t2v~W0a=Bwe+`a!zP-||vtiB0FqX6OHV4F1}NmO=FR%6sTNjx}@D zC-P*0*$)MjF$`2)%{R8EZH8%ieZoLw+4w8_52&Ky#aDl=2VMsDexJ?4iWKKX=MRTH z4G@5T{jJJvJVMSv>Q__(3}iSF-#oFKJtHK;H2rqzx}M{6A9k|Mf_} zyju#ZwT&llXMP_1WHZN3iE8zKwg1cP-w*M7z0`qb=RbN@+(|${U=051JD^Y@IKK=M ztDc5>wI58-l&EE}k=k7EmdVP&Hy}Rvzb^xE5Ne(VJF-85?)A(P*^;sA?g;Jm@mV12 zOS1?~r$Co?c7J-E{QrGOcx)ItD7Y6B#B(Mf5$oybpb(Gn<{E_c!G*^3^qqdSa{up6t$!Resuk>SPcFFGE0A0L*XR8Y zH@*&wPZsc0N!AYFU-9>c0#1UU=`t$@(?cNSY8mA`4I$clsG@O3GZs-#8?Yi?mkl>}Q`EA~T9 zq2jC31!pm?FMPS7Yzre45ssvzr;hLA-o4G1#Y_m%3l&IE5SCZ=G~uwFd*ZTvr_NO+ z=so$vy|~43ZS|Y#&x0_wl&0%|5_oGae@XJ*gn?k2+z8RDx1!7?^wlWZP7*gFyn6Ab zw?Rs|S|E&1-1vI9YX8NzzdZ6SaH!|MGnjvi z>bcShFN?4uJT(1k%;@%qt5U-m&h_P0l8Qy&`wMC|r1pjL%8y1yeXGyryAH>TCyyGu zFL#KKiF2)Ov&4}-_<%&s8ioJdM(HhgaR1qor*8>%5Qs*t9-~pn(R4*9gDY!nui^cF zc^z?YHZ$zz$bshw+@B_hoZf3OxU~;%%6u|*7<{KhY-^|7v1@Z{lV&w^q$?6h_;V|o zbL1vsAsedRs*h}X1CJ92ZetK5yt7JHKmuJf+I;TS#xC=m?>HE%KKb;bS|MxiIHGX& z{6O}=nDCu6{CKYmt6iqd*~>y{^I*AtG5V(H8B{m8hq$nNYJ!MXc;tnqyYw}|%RTO; zfZdB=+R|*XXC%9S(PIPUq8ia2dO`A^UT@eJ{pGA;zOFREHp}UkVFlB#3 zqW_%3e_rDMx@?<#JF(KJs9&b|jL9%}fsstVB3@#ouIiV?qirzGRvU!HUW(z%L2Pul!Qgzgjkr$hYNDrQS7p<*piY$(1GY_PW$W zaAmHc|8J|fn2T`Pk#)I6dCC9(I&C^I>j-wKi!$olM7cYTx5u-Ig$S0*5-EzfLCty9 zH(6{vo@*=Ap?ic#nn>;1F~0-@kwqGDYY`$orR_zeHUH`K<^5bH8@#n3U~AD6Iczc* z??C-KqkXR55$Mu+n{$E~(B)lbY3FoyY zyCMVx~?Df%Aok&nuOJ8Ai>`)yQj6`xqwx?Zl@aBGr?GOIS-`{))i_A%8O>j_!* z05;40)8W&4q;YGv^HCYU!@jZ|$9cTn3Cl1v!q$4(rlF8X$r=@ezQIdX%bPPY<H_j}%dH9KLFwVqf9Nm(Be_S>dG>d7=z#A^V$k40K_%8! z8U?ho_tx76f4>aDJAl5xJ}&+>67p>3)&?A>w2VG0{K@$j;M3!(+np0v<)X)F8+YxfrY2ma@Ha1v)HOX^b0Q0JJAT0t z&~E?Nc6cB{3js3J2szu|@!YQje&CUe`l2>qQKmR4Qjh}b4ANw^QkFS%M=;4`7uSVK zM^>=Xj5TjCR%r~DTT`2_rFe3khMJh?PgUT%oS8+lJ*2*3H%`f~i%z%<|l7rB9 zFr05%iZI{p|3fr)A8A&qA>KL+cjfwtZ2qUKOAY2jt9epFrIoG_{_@Pg%#(e~N_REq znT)zo0mj!>7xTDfs(ST>;ZNma zas1vmYR+&CyTBh=VS?_sRKwdu^`{}7ZyF`3MA+%VGWpjxM=QP;ZVIuhciqjqMqeeg zTiZ)=C`QIQiYslKso7hm9?n)eWQCRVHl zlMXlinYW3nEbH5zbk#~5rJ+FY()c01Tbi_duC3wxXI1AuESo2266@^b;2AnF}Y%*NLcc^eZ^P^)?ZrB z-AYA$KV~*#%Qhe4#&5WlB>O77ipRdv*N(Q&fj38mtVGf12tR?z8N=oF7G3(o3IFp_ zQ~x~UWY@fc?k&V9yyZeIC>6qiv%bzuGOQPdPaaTZgDPFLGTJt@6|j$V;?#q3iFn;J zQ%DzvN#v^>O;p}f3#yYi9(s^C@wr|l#O@ttbVs#{qv6&@1~Ak}J6-O`QI+rB?INM% z{c%cTREO7o#bDsX>An~KBIW74xoK^;i=kiM$MQeUB>bM4u;pZOG2);64_)J`2;F6{ z{KN@VupC^&4|Oak+nv-zncA*`FR-W?iBKodVHgYuo6(jUhr3D_JcizroRYy}N@ZDn z1}N75dYyk+FE;4aCn}iqMb1&Gdu-KyP(`}|@x(}_^G5P+luU&-mAAk$r!?M+nkziV zYSyh3T=<4Spx7%hJh?kO^vy-)EYg*c$)$UC=3RQi61ja)kE^@?W-dc>&;q|3Wlf*& zCCcByi#NY;%f00c|ETl+^S^!s0AyvvuI~p6!Q=a`8TL9|M;oX2D~<&UuaSd$8tSwki-;zdvI8>^Ed*VJ+jFD#TnN zDLaRf)`Vz%Tl$&CPJ)hV;%LbN20%wGUjID+C^ShcYFKhhnZf|s-3|nnQ3l% z=3=~W^08YdFP5NREgyJhgiH3>^jYu6$0t~Xz^OC>BdHwHg__`qfLu>$_-#X!oqkmj zxAyQ$6DL9AWpoq5m}5ue>1q>a1jacM&Ed2E8sK@tn{i$$Nwwb>jg16;7=M1JE}%!N zPb6Yj!huIrApLc%cTTol&#?JKA1$huPMsTVNm9L{7hz4<$0-;3o7 z82xy5^B%mbqYV4;*y)#akkh&YS4QKTJ6?F{Z+ zE#Gr<@*r9u$-XUyLRw6ySqn1tvyo6Vbitn;;r>IJN&Ou{+6TGi*LhLFzMy z88viBW3EeNWhVXu;E;+y(*&?`eDBmIiqwSwoc}!#<&EqJ{5h(VmlSa%>LTip_o|>3 z+hf!$5DeuvsuWDt<5uBFA=m#M{NJh=@$ck>j=ba6|%8QF8Ey~K+K&ypCwM8G32VLkg) zB{z;u>UV;-h2XY#?pkMk(b&zki`Du3EGEv81P;s7IB;?geV#N?#Ll^o*k{gO_#Y)r*K%u>@S>U ziJ|N({={aYmjBFVV=XQCs$e`vk6Hl_zYPzV&3KUCZV8AX@3PK)(v_&OTFTc*dB!?^ zUorNDa9XmO$8E&2J?vHB`U}}lGQJs!g5u_ny_ZA>Yk^_b@MrLN}Ms7VE z>>7obi0zt`h*xcn{{G!=`2iPamNWrFc?mj9wUIY2U(j1EhE4a~a|X&7qT0jgaUv(0 z5vHr#RARzHW4)c#Gqc`&|EsD08oQi`|?O$VC2>?7L#qhSY#4}a?HFIygj z??LG84(YPm!k314jh-aji=U7T=roo;J>Mf)K>nRtcMO%u7w`X~E$riJ)!2tGKY3Fd z7W&DnMYM?gvN;)SHU;%W_BG>=UyN7#+LvObw^p!@KGIT#BbMc?n+0Hpz!;d`_M#+jNv;WQ%3v#vmNn38gw+}+Af(n3pdI&Eoj z)>qoA=mDqzv4cVPc4np=@~_B!Q5VClkE>C!k|&Um9iFQdk?&fclL%F?ZNJO$njd!2 zRnX}6_$rh;TyfM0gVq7xzPGswf42Us!~XuA*sTNfW#7i2-NC;39{$lw@~j}VhkX<+ z+ng_C4UP@k*fMf1%V$9rnrBx>7W+Sslzy1$n~ddS)ly32oM|^7Uk*QUIoR1kr&b`h z`Er2Xq^%t|0k}_8R#&Y!xFfkIvM9_p7`gnkP`c6GIV^o^lOy`SdtnY`we@*H9TFc# zHjOu%{$pJks#zNa3K^;O81!Jso3NMv-n zTyKyRaB&2Hmf!n`!Txl&0m+5;pn!$xBQ&tK9`#HCg5QlGw#_}(17RIt;^%0d>YzR2 zjB#g9MLdI_Pfz-OrjbBwCZi9Z*e2KT7sfrD)rNQiD zOhpugO0D1eerhF?)E=c!o0v8h?$(GcyZ6%CSYbnnR4*pNiqkGNx8uBcug2*l9K&;^ z^#Rmlo`eh#LiLzhF zxYH{Dip<|~cm1l)dhds{;QD$0j#T@C7rt&^0897DI+Alp;={WfEniW%)h!s28B_oO z^_QIKuP>j)&}x&I-^i9qVRJ+p$l&V#F|M}MlFXIC&(X&_PkiMFcWxvGcfK2GzV=2f zEa#1yxnjL$@V|ZqejZj0p0%+_TyYh5!TTi_7X*{e0~y2gNv*N@jO25JU>Pw3!_57! zmLoYrgPZ-XZ80)Y_=1X2W=uv|;`x5!ANh3S^LPu)pBIdsWN>Tx`^@w4B%S{b7ytCM zLNxbWx8%wEy1Ppn=|(^r>5%S@p@xq4^1k2adB64j!=lyzi)-dOkA3|1-WSed6MSeL zkUb0H$b`Ue;$*Eo$dB)UsrBidwGjTlvj+tMmdJJP@Q1vyxdYqX{Ug^de$MYYkciht zku)!rKt;fuKHfj9eI;^^p&EEOrPBAqxLtPKY-*jq7LXYp)Hri4t6XmPE$Nny%Z6t? z+(j{J*rVz3zl(dBF6NqB9uEcc$}D&AB(;FhUW)I2pnf#CyEZ0g#XY$E>%4OLmD*+M zw;qX)_U7HRRy7RkLjkBQ;>#?c`2ACvhL)0S+X@mWpMEa?LaKLYz^}tLfFfh96QcZV zQZHr4kqQ~!!P|;8z7Ix4=0wt(DbL4Qy2mtbwcaSn=HfVpGlTy&x~Blr8xAMmwE0-P z!w$3Tcy3?*&S>UXLfDSk-aX;P>N2TY;f_MqmxJCq^UnPvpC!i|{GgFwHos?7;GPeu za@aP?uLKq=#~W{VKz1OK!hkPEVyKe*K$+PmVuWFU#Au-t++H*GT2zh)*q zJH^p7go)*RkOwXFlF^eis&t~HR*6N%dXqU2K`fudQ7yM4o@wxOo1j};XX&7NO5DpW zHcxZU^_I~=9Riig&tK=p-o)WDh1#f%9&ca1Dq_R)4w#(2ulso~2e}5yVVw3RsDaHV zj!vVUp8*i|Q~b3c_cbnEh^Y{H9~NgbKp>6*)Y^Lhp@|2!EgGI@o1g!YZ~zD?tAgev zbKl{G*jM4RUx$(8m6p;Qk7j1DnU2VY=;9I_0vpY1%HCG#vj?2t7W8^+uxR1;RlS9S!MCGPhJzpLo|$<=_Hn$h{*;WHW_u> z&snD#j%C}3n#Vg{Fv&z6GD>9*X#ei_g58T3!N$rN)S7?%6N%^DFL)EI96tvA@V?w< zFMm6`dY(i+r8)eq{=&Cpew7*y9*O)wOP4RPM&6L?s!TyOp%TSPEY33BASOKl`UAZdPru9Kzp}=3s-p3OW2|-e ze?Cnn7~K~44b=0dRi`{D3S70Etx&A9nBE0g85*7M#N148NcrN4xU7C0onKt!e=U$! z1bkTLW7(qBt-!Z_Rrk7^VVTHu8MD;}+914h&qp0zQJYH(t!r0Drt#KBUt8l zu;KVF>R2547P;mRZBA0P^74dn#6lc%O0$at_?HS&Pl*w<-Mm-n`=Z!ti+TUXQi2%F z-IeOoKcu+{kXcrS(*C_Lplx$T9yFf}IYg|pze(QuOxo8ndEmNYCJ*Pp5rr8CeQ7s= zE6D_>#TVh_6b9cX#go92j;$cK^`wpgsvzcV2xIKm%nL>1CwnsWru(1fh$?jDPoSj+ za4z~Z3wtpKVgeofp+hEUEMCXZuJl=aGb@iGc2 z`^wIswI;V}-%M~N`|~^0bRl~f3SvK-lz`94jEK{iiCQ{cf_~#an?B%vpAGxQ&CQ)c zdktVConHOGMpyn>mo0F}F9w!w08k5j91v_9;rrHq&3vIyj@bQrtZ%Jzp7rv0q24;} z!oSUa0|fz0C_pK;G}!)BNb=DIio$I%6#_u4W_5cwAHIH4&@4^xo*em?ECol?N>eg~ zjlCJa1s3!U;U`X+e9k2ob`@U>UL%f|7G`K(l<77*?Y+0we4=W2ss2jrTX&8(2+eO} z8}c8nS`6LqzpG?=Bm^|;b<~{Hs8Hya-bhV#0~ykK;7Ly0ALAhoQ^g=>iXRrlVLF6o zy+W++Nx9!2)_}uEK{;jLn038!v@nH3?mJ`NJ4oF5+qMF8^7iLQJZh3AP?QmL*MJ0c z8s7Q~b%W>AsDWpi=_j%|iiR)$Sqj0XKJFd>|6#(_>$BNV)IYYMnFt2P*picV#lTUu*%oNb+(&i^zf)e&%&Zk3;uf#b)s%3ut} z3O1lqr&A~zcsXm>J{=!bTitYNR10xU6pqT?c(!fF7JQN{7*ylrIiBo`s`a@0XkUT2 zxbF0Fznlu|3PO{z&)86!M^7o1F29XukhBOe4Ch|%z>dM$mXCXo&^3ZeHqw0l7$_58 zFgSgn$iuq8<+c7}UVrH>3Ji%!GheWH=`~klhz&8VZfCY%@14Aj6)4cV_9kht*)nKm zK6`TSfXV}h6;(OKra{j4qLgY)I<2yj(kXly^_cVutz_dt(=vO%*Iry)3>cnWddn;3 zN#bwFi@f|EpM9oc@!k3kPMnPMg)(ZT>1)2UOlbo-D^G*>3rNi%-momW@HOm0A%QDz z_|&6sGJvZ>gQnC5H*?$7b|v-zRcG;s6*W2L+F$$iw+;ft3oQ+{W(PsjP7nLk%`RI= zg5K96A!8pt7^gbL3mK}?X_mBt6wMx6%zA&vTxhgjeZwuC&5>C5B;&3aA0Eit@_6hH zcC9u0Bi}g#e+YiCRAb5=!XKGn?>qyFTyWS`Nu;C40)bgeqDMw8AB2I?yTtJ~=<2*( zM{95WX42{WR>eUk)t}Ay52Q7xG$zdQt7e@MnV|QD2w8w-sHMfoiEi{!$mlTjI3A6T z(O{yCx6K*Od$E)LN9@GmIYrsR*M^hDk5gX5g(JwP(x7797BPJ{;VMQhUEcF!lFeU3?ZAsG}OfX_#n?HjKSPMbN=3r2==4KBT9{*yTje5TfG`N4Pa9Z4CLZ1tZH zG|rm)aZT;n5>|+%pVUH=?+nkCwA?TpgAND4HLRsgKt^275SB|7bN$Pnb7p zgR6FgdVf8V_yNGtMEnP(Lm_)3EP7;6$KtZlt*WM?YNaYLWeC z_bT!h^csmpuSAdPOErR!&wh*Zja6d2W+=K zDs>Kid0u3_l@EWp`hD7BGFm8unh1&v7u=H9?@pDkZS*>zNnlP_nF&~}|AxRQQEN&L znu>jfj+M9J^JJ0ZSNFnGjZ(rc8)8@_V=s54RsW6V&GFJ_>r@>)dL6%B`STk^65Lhm z`D%S*o~-Gjg8R`ewTqBQg}Y=J{)FX&two~|*Gub>~al+2zHAPRNKbdH9r%JWc7c1{V@o-D0W8ls*8*b;f zxNnXYB-5<)D|9#C%TMJ!eWKH=OKe)IK3vvO*q>|X%@A}RHXxNP6&!yy^- za4u8mT3%Xl%Odo%2h9YLKe-gBitG(0+Z#V}9a#ZV6nDuRAQP7@pC;74^9RzD#rjf9u~D zPAE>w?~)JLB(P@~3A+S=BSwa5rLNpbJSXk5?z9Jh@?;yXZqJBH=4msH8fHWv$IFE8 zO%`!Ob%SRCSTo8PWCBZCZgFL9YH5*Ks|QY+Y~-x-0^avLqGi@VO3v^czrk-1ui_8D z;c}})R7U0L4A81SfUIo}_gZ}kdaoYf=YPU#{MP9Y`}qqTbhF+QrrZj(|Blsd*Ts1T z%ButsM@0V_WDzCYkd)tiub2y-4^Xejn)3YT`nfzLR4G%B0V9xZ;>m$@0iYkI=wDJ< zbAim=z0Z$%uZLSsAR9CdU@pZ&-%`httT|g-VUh`Cb6U=f0Sa8cSr$Ybg-)-%HF=R> zq>;|{?k$V(Qu^utqYFfXe)(>H9vEN^N8d{?8`PK9MSxUxopir<$k9`SIM!{WpAVA0 z`HH+MS&FO&N|eG#cDK|QS>URe=}ll)5Iv=;Tf^L*mtT46ShAyj8)SyeG?w zWWA$Vf(5;GLuky_%rZ;yTN^=$Jf`xeqR-_qGbIld1h+(2NE z1y7ct69#b%8l~(l`K@=u`*sEOcIwsj{S=*E$cbCE$x+($_uP2|n z!O##)%RgM~7Iw^TauIjA9Sy5wDlMM+jDtaQK@w{WSe+tn>RKh;Ti(b-Gw-SLTqMH5 zs*lm(sA^DAD^}C%agFJ5=B9Sca^G#k<5YF}jj^$r-IMlgk!M$lD}?(RJ}c|aS$AGv zwocfWJ8rM;;jiDcIc!PKF(7}ltT!IQBXmPGD5Dlv-}ZTWnEiGIu(KZkuur+(N`r{a z2vlb|EB`&Gzb9;A63`O%Qm+Sb_n4eh33vcTrkBl zB{$nh3bhQLzet2-p%viInusIb5a~BfCw7a_2TXJcxveE()W`;QzSoh4X`Gfcf(<|y zbof}%&*tXlLAy0}li_HuXO`(NmKUtHQLCO=Wuhhh561@ra$O4}q-nReqRD|9JB|8lOX8FM6FOP(NwU8h z5kmn;wAtbSGVgjzWBu3hJ4VJ84#eK8>&f8hw_-r8XU+6JM?Ur}*+7ymLxpOQpQ@07 zzh20uq+{V*f$eYL535-2tA~v9Rd?^o7o5{Cvrbt~G @JKOk(K&~e^%Z}@O*1UF! zm#CFy7ld7OFigv!lkvd=9D~DOib|U|DtZ{{h-}ZmdjYeCGQ9#$ZsTKVULiw;zXR=C z-*UTjyA7RZj5;*_f@lRzZ%p{yPsz&vdiuJMwaLoNz!Njgc{VZK(p^<~1(8+z|H>K74$^A@mBf^$sjg5_D447cNkTwd~om&T9cWYx8ekZ>u>P zd%33(1)cy$V`(nLLZ;YZUdB*zGTe>cJIn|5A7~nH#!g-EhpoC7KRozC@K9ln9 z6FzcbL)0~IgMnW3BdV^vm01@mS&rknw5zNHxZxu9#TxunSJ3S+T9ur>u}FkKRd)x> zzYU}T z3w3miC~+y(r%$vvUxiva2zV$gXCq0<4A!i*MS<`#$&OJ>rGz}78}u)t@{-^~a9F^O zfcayIH4d{VWfO5`e+WnRG50>f!&}G?2#JDTZDOR3n~0?=Szfol%&1>pE$W9=ZU+&} z+p@b;4C`-{Q}=|}b=%;5P4B{BaFZM968=4VTpTf~bJ3*~l zxI7i@Xn`uJhsDwPYiv*`p#5TmstKA!-M9iUc^b-<5C!Ue3E1=D)_G3{E$XXfP^e3? zKzKp<<>P~g{|<{7Gkm^I#=Y5~O|4S5765@QmRp-dmHPS7pSuA!I5uHs0fV)S@YX(X zls+KV(1r}2uJ=$uI}Av5e?A-oP~3phq!7=S_9Xt-fM#Ia za#9<_a=5-Owks?U)amrERJ@iG*`4bg*~&*PO*m8LfBASn5fKWTa1h)5c|23Gnfuk$ zR9+6TvCI(;$h}wR8`5)yY)@lhQP^!D!yqQ z*g0f5QZ4rtS@kX^&gB}WfwIPLDo^ymRUGP@gsaw<|s<7u@`QZ3Ry zuZMjpb|0m?w_EuLN*P=oK*Lg)Th7n|>2oDam#jzxnal*g3?`2T zvM}3iB#Z}CzZWs@y$Y?An|g&yUU#;P>WW9sH_>xpOT0G>ctr-I1F_(SnG_lDDos{- z>!8EllLU|>-QU8mu5mOsYm~QcTbuJx5EIJ*;4Fx{7>}xcG58jDttL41DvAwyQa*)R zRie#k-)8?N$#z*m=ME#zE#_1xl!%j$bg@NTCGNp58);Mz@d8EQ6~*%Qpw^dC*O?IN z+cNa7iDBx4H(e4`!n;2P&S6r{_9}l49n_y}G~FVaBR228WLe@i!La-_L;V2Tq7>7Y z`c=_qaZ2DB9l~SB4O1pARK>DOL)4#2@SBB3$yZ_(-z2nr*B-(K=)r<9quO&|EE95J zh7SR>M}L4FGEKzWIIRfiYZ1~^mZ=M{LA;i4dmw}mEEj=|VidS3(AN^zKDPj+$SGO`qfLMXBf+0qAX^awqD1&;EBRv^ z(BDH$>qB51&%lgWA8&y0gleKQvLfjOX(4Vt9-e;Z&8s@22FXS#)Bk`3y{WjQHnO~e5C%4NMYmo3}j z7d&A7Qe;ZxS$1w9G*e9~r?5*3R_LyE`U1MzDSCT(M4$w)cN z8N*-irQbOu5ow98$$^5->P%kg#|H3D>TwU3sRr4Nx!bpWZ2S0CUG;AIt(bOPV#ZYA zz+1?-(JY~ygoeF!ZgWw39D4BRMOXW>p}JnwYyjSKm>GPsDM7U84ue!7xnKVe*WQ_` zp@YOm;&%5e1n1%j##4qawt*ZlK3BTO3s}bDFZ89=CM} za`9F7yVlOJur4>d#OVlmt(gOt{%l_uk!| zUXG5IsrW@X;$_b29ca|bQG4pN)#(3`UcV~eesQy0qb33e0RpoKP#Z1-c)!@8Tf-?z zuWJDNJUb7k!}LDm*LN|g;-J=k#(Val4SnzzYgXJ5!rYp+pi?osz-$|VWgzcjDAc<@ zr;IzKhNm|*M~CIt)5fjK+7{5QsGA}~;2cXMCk+%+(ZUWYRLhC1|5k~47qZ>fPU;&MP}2N3yqn>3%YGhJZtu%Cu-@D5cXwM z>sYhyx{Kd=1SWm^D6Fla854@GYbeQlZ-q6R~XB9 zBD!;Y=MeUCYT_^tZAE)#T|33$9;1C`m{@w%48E7g%*@)$;H00B``tGF%*{1#Irn-S z^&{Gk1haLf#IvATGG^!_<<+r07RpV3r(mjUqXp(#;VU$_7_@2cC0^ai{8{~uTtIa1 zR_RQ&st)vQX5!4Xi+Cnh^!-ReW%S$+6M$+s(~co$!7)=n{ddFf&Zj z*F_p2p|(yv)M#-(h>hW1n)=J9yq^96-0|j3(cwC>C%$Ws`RG~nmO&4I{+pbQGBO6cGOD)b`F)yA+-z|#o z^rXN4nI*0L{xT_-#Wxw0msxEQ!4{XQ0Eu^i`f+tR zhao4~6Di6(fl?%)w4k(|xqX~{O6Z}wd4yg|Z{#1E8-0Otu-L?2XErVtG>nYhF&riv zf_m?T1mu0Z1F@bUAJ0~j1MXQk-1WN$m=Yubjwwv#vX%zA;i#Yk^9EqkHDfep9RV#v zwlwM%S9m7+20DBK=pofAG5HdCG7J(=r-@jR;0t(d@ItrL-Ocx z1HF<1u_4$+w==zkR?p#d{|p0o-rsF|>xZbM6l@)+F*6=jYd!+x5#eiBV1XbO-TD#y zHo}or(?W##8SSS{i{eyt~x)i*?okpbyP*o4sxFt`de*PAg0)Omdy+nNHrx{RqXrtWQoc(jNI#HW86Qb5EPo=v#YOE+ z6@D3tO)H^U!lK%v>MYP($Q_Pva1PW`A$ox4@x*i*b*I*JCn7Mo3Rt2efN)pbGzNCl z%aoH)cR*~W&6gEf-U58GgJb{v;j~aKL#+ZGHuHDL_`p)#pIHbKFlYa-&8;&~BvRm| zQRZwBr4b96s*9Bd?YXecWS|d|N_)w$%En4^o1VaV<@cV4eJRf{+~Gr!kUu6*$_Y(a zh?Y}d&`+v{llyR*-v>>Ty$7zA2Pd`4K`h0O8tjb94#|xRKucQU6~%K1?oBLZWabU69BZ(6b03UgCyum@mNnF z8vYEtx)~!%1xEGHsXV+_S_xovhI~Gct>~EL%$OkvK+sVH!l}#wHzZqB#Uly9mEVh3 zX24*_;e8L?iwXB^qGMn`yY3RY`^Y{?9_HEM-VGhi;FkhIp|}Pb&_pkRi;`xeF5dlf zUDyLqV;FzqZ#nft|E!~~sE8J#$J)-k_AubIA zoDPnzGJ4MQPTQD_>iI2PjWcCu5WcCOM_r|sm9Vw zu&AUYOC1eV0B2T@(8Q3onj-QCiE2rOH?PAt*B7S^+PGNz&0pWd=M33~RJ^2i?G*~+rnJ-yKV%JomsjGFU>b@uSJW9hy= zs_iyvP13~0DHUiV5CXgZ8gtMfA7J93^6$E1F$9V3LT$?Ft zyX(n=VJT!kkkR`68PmskCGk0X*{u`HK@L16_>o zL~|#l)`KM;Q;sFC5{7Z(65rGdj|Nincolk#{l>le;n?;Y&?<~a8D#n#d4Wgi$P#hG zLeWJ1Rir?Y!$^tGB8J|5BJL72JZHuC!MiFr+OMjX1?s7ReQISfC00NMFo5gK4us6w zZ94bYHrqJ^vvJef>;wzNhQXT^^!TGGrMU?@JdR(rxOI6Q>?&yWdLF~tD37mF&|B95q zbu{;i2PR;w8$~vFB`*TwRoYqITA{A8omEX`G4K_o4T{M6{Mj0$Rv`?^BBXwn#qTB7 zF=Rx#7K7k5OL!LUK}*B-{yE{%?)fib?);}k{VXjua>!gGnE;&jj|LO%I*08-LcO(S zsoS_Al8hfE^&6oT_f8yZugCj=Zyok`+x3W?AEyrp{&tTFohFf&<(tpjJx|MK5|Sz9 zo}GDiqu+LAnMYt%viN0D|LTH|#%^r*zkZJAA268k+VkOlsjCv$ko^^_g?-`!5=T-x zn$>?Az@odr5)e(wc)M2UsTnYJ?6agL0rU*M&)khydX_o;)*>izCmCQK47>mRa zFq#i;#L#lPJ7s+ieh+M@U>7Kv0&dmAa5Svzu0XU2ErQ(BfjI9D(r=z8kAc32Yip2~ zjP!-Bf<8ZKo@H4Rkc?6v$gD zk~Y-6$L%}t$>z2Z*7?}+iWKy&4-?K^k!#JIE+`3Htm9YI8iO1FtViM~NsC9&pl@?` z=Q=qzgfC%cYJ7v1#6tgQ(Bb1>Z_&NTCQc*X>66~ScaTN;w`M6@I`OdSO+A8myMnfh zyW3GSg>(7j_j2R>Xba}@=q1q^zWAc5$>&YCr$^E~N6FG#b%U6mQY6C%6py{pP#ljAEV=(Am^{kXuTo5JBDxXr*ljbE8o)2C9~RtGUyQr{hU zC2)5BUUkQ8l%U+q3K6)huqKYcoUV^4(ykjXx5h4CguYxJOzt-tVHrooB8)E>TD|hh z-9cOPnw@!5k~;8q?OsK%x%M$@z?^TV@yUftzA9#eSri(X*_IN`S{!lx`tXOd;6F! zW&z(dMzl&3P&GJ`wZ7VT=qvDiDsW{v+oFRTZ0<@z%|ONAC@g6rUT;&tBLE(Zcq3*D zs0!wQPiCr0U;JjiNc%i*i?a(9jvH)vtMw|IVuwGH5s8X{j;9o zOYOVjs*)eUBu)Py&(J~apW=hR$gsh8|P1&e{Lf_!_8O9b+DT^_Vkeq0Kg4d zUu{k}LW^j##d`W@QsTx#K=R30g}LC@oiBZ>j(S``$ zyP}%jV{UoM&_%-f}UhyG&uN#-Qi*T;^a~X=LBqb&j)z z{Worb0A;}__#M93l(%ARPr4&6q&*c3LTBCt>VXFlfWPLmUn7t)3bWt!k@5X9aktm#9V;3>6`F)*LCKzQkAg zzgyXAWFT6ENnEsn+#}#pc7U#RKum*lNNJ5#z%G(Dqw)SbnZS*N`65ZN_M`e#wI*eN zVTaF?Xq+U$o}?D}WQG*^2aVj2spgNAVbwE zU4e?{;jQH4;p>erT&=vezswRmrG&W((Rt?wuR#X2c~)gL9LLt1CXGm; zfyM2=hzY#>OqV}iNR}PE@%o@qUww!+XxJA)E9joE%M0vio1w?G==7SJAmWtjt^DwH zZT^O`q(2n#R(YCRMvqH>?bG-6;`(ZVT`-@!S>+>#lho$MQlm)|L7eq0d*p5(u)Fy& zC!E0f950uxb&1VnME>+GO>PKA%3234J8$jkba@xR_{$LFR)} zQJmPh&yi<^FmfDj*!Cv~x^og^@`tv3yKNg{HVgT< z1J&_N;wD|l#I^Q-vFfN%`{wmO(W%Z_OBHx%u~(A(mAdXj`+30^5YPJQP4Y!FeABd# z76xbni>~oHw8ED1^5eU|$6vr$4jM;0!8RSA59A$m9y;H}3i5M}17?v;(-xB^G3B3{ zq|)?4Qp|xxRiit**(LHHz82tK-3`vyH{n7zR~$M%O3EZ8={HyEwSV$(wHfc)+#py? z!iv?i1^2s#dGr)1kZ(EnnQ9_ligC58$&&y#8JZXIZ&w7ZfZY@}=tE9s@JNfCEBT!q z1Fypx^`E1-nf-#NlRaYRlwEG#GD|PD!Pvopeb(n~j@^Dm9M%z0*Rk(JB}x<-Y(%oy ze%DpPRsr+qkaqFOlp$IQG7{ZXjUdO;;v%SW#TrceBCIowP(K@ia!Tmt0x{U&+kKOh zdvqk+8tK!JLk{TqR}C7VUh_%Kr}oo^7_f3De0CSiVMa#xeTnD-_50~Ykgv7|xIU6S6*6#2(>xpk;ek!=F{^Knvo+W_|I`@sy zI1~?0yYD#c<5y%2r^&ppJC59xhI~TDy6{Ib^Cd04ZdL1MI$*7(CuOis5Bx4FUH5=b zO&)hNA9nYKe(K7G`0}3PO^JQ-5Ofy-OK2DrGU*g343tU{+AvNCP4R0_4xQ}5eGBAqNKiMS21@y`VFPl$(maG8OF$MLr zyygDSj>V>ZPaNZrFz~B+C329~QJ)MTfXc2HgnHK=JRjLYps1H?ZZHt*h zmtqIoDYUr&B_$MD?E8u3-h9QvQ~Z7=RKCaI+AU0@WM~T#2T0D4^Jc>-Khf(qO&4#Z z$at&wp2mVWq4bSd-?;NuT_oKE{_Vf~z_Q=WhKz88wBHjk2zWr+HLEj~Y@|=|Ydmbr z2E~EMfPzUhhgeG>D$!#;jbgKl0#F9-KfP`qLMbZBb95MsnVT#4bZ|4ql9YONz&z!3 zCifi(-a1N4OmLAO6x+ey$!#u3--|UfU94`Y{sXw6KA$$Yd^Y7{5lj=f8q6WwY~SK< z104Y)1i@`XS1?@wMlavqvNu3NP)ON%|8uyE)j-E4iudZZ^~-*~`S%$CEjmcM0q zVCE!K3e+wuzo)ve9j<+;oM%JjPw>r(2QIP}+U&HUD;A<8gYq1cmxp##=flhA8}27& zM`K>T(=6T9%u14a`wvkR{zSyRV?Mv%X*TOK*Bu)*G4o*LS_YQXNmh_`53pG*f?pRh zYg(>b7mlEgb{MB!j%!1t#gHFyOK%hm=UsM@M9l)h76Z}n8vCKhTEwcGzn#?x;E<*3 z@rEw#%SFhn`h)ZTCkr3};BRHwe_y*}kQhf}s~`0t{AZV~AU|A=_=iiNd67SBVbPp6 zy#@h`dAH3c$SaLkIC6T8i4IB*X*rm=XLvLOAIMnrO}Fm^}b((*=MhZe8xRDwW$R4vR{HtG2;q z3Hp?81=1Isj^)M;lV3x2@!OMBzn^w_{7qkTZ2RaL$)eG}=r#1gf`fO@=(Kr>jV`c> zOIZV>)2Xw>dSXZ~p_saz_~aq*;Z_;RLO5}k>K4h?+|es^9Y^KzWt?yHYrhCo7za|P zAaGNdt(io2^-p3Z#;*D(g4%h4%%uDyx~`;^IRyK9GAA>Dhv`@d8w3UdF0UJ&fK-7K zYZcRtKJqsTZbm#W_&vlvrLuH=`pZb%Ww!EF^X+TKWEBb=-EA$GoP3e&FGSfQac4ze zROiN!j1v>aSj1>x?`Vbym?`TEn?p@(9!3y*7CXywIOws+HJya&J8nodXchDe`m6Vjn}!j`W~ zgj4s@D%))XNt*QZ#wm{HrP-IxKRm^>k`#F&em{QIu3}$3+ax~NDog8#ZF8lsMry1f z*P@;kzOIwy|B-dSHJ>pVwkw>G=X2d|Rbml4Bu{K1cY0q`G^_r6HKhn6X64>NVj;P~ z#x@p@V#h=XUgje??{9Uoce#+Rf!(dHm2eh z>or<_5ee{pOq;1xI;!B*`1K(7&FZ?JHP;XMX%snQS*^yJv#M~?Iq0@ax2CrJ)*z>h zU86JghS%Yu;`z}{!_sNprSaT4^B=y3Qt?a4A`#uj)jX!VT+1K}yebq1jxS^OlgVN#F z%clq|+w%It?Ej36#0YKoA$891RNtu^7h3C7@1i*h(v=lkNTsdXR;~W< z@_aYSw^c8XXdN^Xv`DOZd_ZTPT))-YyME$-64$uW3EXYY+xSyEN_35BI$50KtZBc* zVX5@OHHYgc?lFVmro{QRMO*tk?GvIRvoH9*?4YdlHl@Ytl(Gg7@%6xG$52+Vr?4GrOFUc2{^a6mQ|-pBDf0{+ zc;87|ueC;KulC@!Bd=iFM+)N{-V2V45cZr(j?RKK+uVl+!xvNw9>*k-ILEC2lqAel zBXe%od{c`<@b}8Ui!d$gZh94?MA`p+E!gtwAIAbmzN%u}6_bbdU%a{vWw`Eto z-3p$S<)4at+t)Ajt>M_G>w_Lib!kYNJ_I9p?^lOO-%Bcyp)FzszV)!!TR zOMB5=J}G>_!YDlZ|=KOZwEq*uD=A*|*|Hr+H57((utl_Z5<0na-j~*`~ z#BZHNNvsAK+#O!;;$NmN5$qL;Rz{}eJvy5|+y(Xv)J0A12_eR(3pEl`KjZHedW{c{ zj&S{1pO#WfGdbR)nkuw(-3Q*z_48TR3b{A$)&AtusrmO#eCcI#md>=wVyS}Gv%*qC zs-;Td_kXSLB@VDkFi4-`1KP=}?k-mzv&xo#R=~+jbXF%vn<-Wa7}bqu^y>pUZoPl+ zvfko2ObR(Wdx>WbM~35$kyD!mwFa8Xc~;tVcyHWfhhg zvC{0m{5CE@(ehdaO4>7=wI*BpwIier`UOWtSmh-)^%&-6RxwAM<&{T{mve3X?t7fh zmt2+@9e2NsQtl=+YjjU(Zmx%{&-P!XmGwQ0D)-a7!Xc7mk8brx(2xIKbVJ&FO{fm$ z$C}%oE{WL?t8gu)-P?vTjYV!G_JC8iH&3y4&z=5~w<=*%4$^5t} z%ctS1ke;qJBldKMLOr4zJ{iTBFdlHuPeyju<}G}(?vZd7)8wO91x-^oAz`QdL*fe@GYm4~NM3tfRBKpnSng9hAYjR6rH5=>r8t=P=dyhxFmVWognc;mzZ3^zN zal29@!e4QltfOqn+bi$tk=>Lz!EO81Dk=BTP?1di(2Cv)MO+{5z>4inW=cVLL?_G>E&nbx=#7}6|b7~mhlD{GAILnf=h9i2OV`~*fc`q)0n?Ja*xakWk$zQ>; z9at{%c|YI#l#1{P&N#_*W-qy0Vk#nOx7FjFkbPdBCXf4nFIy3Tmriu>SAY0&nwLv# zoa;ExhOvDTG`Mw*1s}Pt{OVULP0;k5Zt5KRF(V^Q@y&rEvj1E_vFfXb*h3QRjz>q_ zj_rwmOi=Uf8XAO`C|@P`OfX%SULpPucW)UN_0|Q9!U#x6C?$=6DBUfgs5B}_r_$Y> zV}XKnNFx#=-5nl4I;6WxQo5PDXRz>`=ic|j{q%mBAH)1(?bWsSs=#TFjLnSG(aTX) zydj}oT8rX5fNC-APez1_;)S)PMX$&=aTuRCgklIhm3y@w9*r&BS3M?&5(=?SmpJ8z zskoS`i@wR~L`EGQx2|cMo-~ct1n9)5H6K5(DI1cHV*gZoDE|+vG?3ULluEn03-jtx zJ`i=D{D>4j;?&nw^*i9W?XxYcy)vx8Rvcr|ID?-Uy8H&X-5d;(HVNg`kDc zuspK{W>%#iwc}MITDH@=SNz5W3P!mB3q+1Wzb2p-f3k3Kb0sve1*`KGE0YQ{m(J5- z)|Ww%>%q65`?SO#$p&xgX!U>Ck)2^Dh!&&%fVwG6IRx_-AG!3#Z3D(K!tM+n$ANEM zMsB;%pf`Jwn3-MFooLA1Q|!dcPPwTHR$*2$Vs364EF)W-Sw#>!PBrmlsh6bMKK+K4 z<+q&&ysc#=tPvK5L)TAg7*xh9Gy?|Ha<&;dOwA;W`nuiR+{nshvTs@US>`Vc(tPz? z3VGmkVo`K#xhAb*X#?2db5!-ocKS7|Q4oGO^vfmcp3S{~2L!PrZUtaZX+`DxU% z;cmis)JUTjZpojy!d4DxBN!0(atJ)ZTEq;aNdFFQB!kEx?9d zjlVnD&E2}}D6xXZr=ZbaS@+vA=MZQYyr;7A?>Y11=%KUR6Dk&I!_~E@I1!Ddl2EOx>6TWzH z*75?go2hBniNvZ7&7o>d+2@vC3Ss_=UH17ZG-|GBoA#ej+@daF5p&;2 z7ayj`=JR$sam{Qiw=QH>Q!-2aXzQDe9XmHJpcEc&JzvPlL|tjBRkhQ)i0olEJSJJ3 z5yp2m_K!}i)&x>>{p2EDqCqr#>Mq=-Jti!jZ@b(dm^NT#srBaD0R4x(xT8Is(i=Fu z)i>HQRd;!J+!|J&)i{=ktcxUxoh}?*80IvBSw=W^Q)vmc<&^Wib5w~bdbYaiKO-}O z>a}X?!76GJWZYFYgZpJwkJ8z3!ThPHXg2q(@R5=z6{{oCz8_)VW={*S8mnnAF*_v!F^XL@r1AYQC4)znNgPXgd%&`W}3X? zaUK~;t+^6!oNBh4pw8I4F=S|wgh7jyX+9qCS?7qHWgw=edFGAyxa4r-rawZ1{QeNj z$65@`^>Q2kSZvA4A(~Wuw9#j=E?)ILdM;e~ac}Xe$lQ0RMb7i8{pKkluViJ72^up; z6pzSg9=DScDp;a}pdpO&$iXwaH9CLDb7I3~qb0hS=H$uV($3Cps&ekfCe0m}X4sSq z<5hwdA}-u0k|)k0{3civ`T!;waNGweJ9ux^c32mFIV*IupF#UfQo3Fuk<@A-<&pic zA7=~xF`*LvzeF==DDEm7+=7qqCp%D#Pr!CG4&E~?D_uP_TMRc_PPS>}ZW?}O*y&h| z&4sK6vq+DE$`~!H8^)jTu`HXDakB59t`@ah+%^z~^RObhS_fw@6XcWqxbt*mJ{^RZHaI7v^^e5qp0#*3~U}c1Ijl73HIr(Oo;2hVEn;Z zmx%XI?&-CKNM`8>4e%8=8F;DE?>=t&Y;4b>1IuC;wa--zWZEoD_-wYk+n;DAH`c-9 zgw@8Z^WJbae}u8gN+(dOy3EuFZ-8}qtpKKKA2+c_IjXT%uJ6Ah`d(F29)DwJq<6OG z`pDdX6HDlYj>EgWN=5Q%c#QzExg=;=s<(3o9X&@LFnu>>vPY@+f{ zc>lz;u21dDv`Y~;PK(Ad%}u{L;AWzmC~5{>KXEf^eZ`VRl1t@TCKn)-tJ5?idYJ!d ztoo4lo|BwabFmN6i80??#ji$4fPFNQz~&H6}95syKTq*T1DkwO@=8 z%NOAmt9~k%6>Bp?bKve$q_i`VU>QE_nzMLVZ+<46+mF2&65SY^4$i)wu`yx|KR+xW zHCcuW+ih7dv!Zf#!KjdOt{JB_8MAu@2G1=-xU4ok9%Pw}6pgSy(5SVa zO{{bzsqu1kU9hpZvEsb3eh@PUd}9h=jl8TL-yT_R)YdRc&9$7+aa&*v&8nP;qH3m0 z^UxI*(e5RUJ*~sffctl9f$w|9dCbU52Ga_K1IfeKU}lidyQyXPC%SG4ZZ|%G_~2=%EQm}YNeSD<@&;lz8*(S!ACnq0VOcIj^^jH4<0Br&0EMR ze^v6*WQ|^R3yQ*2zmQ4g{^&Z)+3kb*P(L${&%4BI55uE{JaJ*$`9cL3T?sG4lN}A0 ziOeCleGw1+@7|hUO+HTS>&uLAe|6mrPs8PMh?Z_I2L2M)Tl;LntNnW_qZOL*ZfOcQ zvAr^10-4!Y`Jy~6g44!F6DzV-0Z7GhR*@TLN?i=h|MjK~pSgABbpI(NKXg$L1a6^U z6-czaEOCmr$sjc5wR#@Kc)-S%U9{9AJ5fQULs7Vi7&uer=s8$v-I32gH1c;z}AWDZVOTNETvOD)uh`zbD`g3Q=@>ku>6P3527jwqjTMmm_ zPZDt1Nvb+eJdUASZ9Yz5|xmn#9`2+WpA==rEm>Y#z2IMRJ zKa4TDO6?P(TwIQ?_ezy)ZXuj>PRw4Gng^uH!omJpA0Ck0e8|1SU^BnnTpOGQ3)+Dx zc!FH!A#T`ZFh^P%!=H5VvKDTx5LtrW*rVyIKx>zX@h9kQtdXrYDC;6JUL#3^ydkqi z1HCQ=U-|kBqvr%kIN4R`{kJ7;ZY!%+^*gvmMmso)YG|`uIMi=V=6IPJ&_v(RYd*=Q z{J6DbdCA}*Oq0aqS;^sOoZJBNX?c>oa!yVO=d&U&h zt!0nryP{Kp6aJm(e~|r*b{SRvu6Cj8hpg6%Q}2)BZx~1M{x_x1sre*EFoye6?Bkz_lm8O!v;Wb_dm~v- zub9BKnemNW&?(}$?(+P%tMl)_UNC`YYjOlgfHyv+V+VuRqBsrSR30v6aF{|(fw_bZ zV(^m(3@XXDgI56gdg9K#Awa~yxd5_ovZ(*;llz&7ekA{e9_kbnbfmQ2xugOx#C#0` zJLvS9{rDk(1}}DvIC=wtRPY#06yKQomr?gYbpHzC9?*an`RS)+hH}Hld4C~}d=Fl* zgo%ePKV~M;dAUg+xjM0JHluU!vZOrJy1?~-T&81LG6%0oA8IYy$T{>TkM^gXL0=)D zMyl+2rUHk!8v{I|T;`4s5FdPGL+Bccj}`z858M^`kFdzK=Sh!H@S1KSi$roR*>b7= zlssxv$lAxgyZ4YG6P`y|%-WUwiKE}_rElMcf@y?z&@NrY%N31B1?|egfg*>Jh%**m zVf%gj*3u#!CFz2P3=*TdR)Ct*bAB1` zGEWjEU}u=dLhdwbvsqGeB6o}OL0aR>;RE`N2oIm@v|Pm;+a5EZO`owRL$UV*Mc*36t%Q0 zPM2|?&PYsZ{Qk!1Um1CgL%mw_bI%A_nKEe}_K z(dX6XYiaI)Mogzn>?mK-dNkcj1%HKH}$!SSt`&VPEs`L8S`i#cx%fJ2e z&mDo?LA|V);43QV%&z zc#KfsH*&9Oo?>}0(a%-_vo zXtnpy#qQ9yX&|%(0JM}X#4~{r);0Ii#EVM`Is{u?%iIranUFG(XSX^>g{)vF%Z=Zvs!3NBw1q zCZ?Q98ZczNV7R-5o7(+owZu=JB#Wd=P}tGVuE3vCSbf~{$by95I2jg7+5V@)ekG(O=gu@P@^(N$tdNfKnN437iKt z2Y4fFyvj`jDle>hAu*c5{6Sf!TK%DCQ_wYKEudS{36J~A3)_c>)m@geN-}GgcLsE< z3|bHQTswvLRFRe13;XnV9)j&|I!q5MO?v2I;}we`DT`*4IwCH|8{9Q8j?Hj?Fhp?Y5HVbG%mrBP{CnLBr*F zcQR0O+9wzQqYg5GI8v)}&A|>jl4Qw5X6_+DQ7~p>kaB5nTl6pkS?_vO8;XA#sIX7I z{ISUmQ8cDY(0jeYX7SsW&;=qPPx5Q}jaXL5-G58kx7%eB>maz`b+W9J^jS5ocxb%X ze24~|u^F0hx}9b_Y?3CGqg|D$b+l3tNN(G7Z4-d{n(~7pnb~}xn$^hP%Zl06;dKd>9V1Jpi5jn!xNK06a~g5GCg27t$Xth$>qmXRFu)x! z9Af$BHRb&lxvNQjtXJYX_JQsw;Pl+Nij4#2*Prxrw?$viAgK{lDvVnt%cO*H#%-#V zfCH5eiJcDB2ZphWOlk#5M@l9<&6GZvZ4Jj5-VoWJe4f-<&YO#7UH7e|e`}+W%FpoD zGWVchRUukPT~^&1ZYO%OpEsJPLyNr@t=r@AitgN{*Y+f&Gr)q-5#MAdsRfz+bgPx) zi)Kkt6`%4{%ej6;P6TLzk9MTJ8f z!P=V3n6Ug&5=rN907YzH;wpFq3*2JpoNh+;hxL`q=k88)Tv_|%88@w-)}lZ|{B$_Q9|=n@WMt1Ho^} zj*TX~8-N^EsM4_SKB9#q#I1a|qQ|+J+EwcE0&`)-liArP89mx&XgP-bBN~kixvQJH zo$3w`O`OF9LR=(%9SXS-y=LX>J=`O8nUIMEVT*D^XUv<8;4tK4cF`R8j zyK`=8@LFPNI?`x+BvJrhR2G_at7Loa_HIsAMCZagxzAh>^m;XF^#nbeHQx3gE;QU8 zMtO5JABH{}`>MMFEQ8V8gWNitg7{UF5S|ZxUK&NF;gz72tQ-t0FDBjo+P=J_-g!tB zXRFSo%TwnF2xT#MI6dyR9=Z4+obM? z-d1zd){hcgml&8b25QpZyumzJ5QQ@>fv=La0JqEbg2ve9p1VUv z{oG$sQBf6u$q-!4laI<=Uri4|s6|EtEk)HdnrdLYXQ%zxBEW7})z1SUJbEq9bpqUz zRxltqixYNwccus}qP4X(>y!PM?&|VIDtBtqN>(POP*D)bJTbo+`k6&DfN2gR)D^dQ7bq*EpufOw83iV0XHoqQpZy*yK+;S>G?sU=xN@^f#sYU zOBdr}I|G9h2mfJH!`$I-hyNU{*LZQi0u=$ew@4&4Qg#fB=n(W5cwL?8x39b8r{>ES z_c?f}y}|2y1kZ8UGX@zws}Vn%+cRET2*6IBUKlJdBFuMhrV*G7suRNT*#n-@EF^EA zTxRoHkMM72@+u}CQoyxrtsNgK&zm5F!ChN+v{un-(@1ecMaQG4Xf10^bfX@BZnKpw zvSO`VMqc#9ewhG!b1AHg_P&46b6ktEM(nw+kTT2 zp3+QQi@n6>I3W*2PabENFEKKyW-nCjPy3z9ZIMF52&uovW`DC`>2a-N;C`$2C zQ+vCGkUJO!&S}5cRqQ(A_#FbJBPa#__T27>74C3BJc{RX#KaBkjCgc)wwLJj8i~eB ziqmc1@YfAZ*f3B`Nuv!`BV5z1%%z@x)QodZ)=P`dtm~I0(V$=zq3>ba$RJ~oi+W4Il0=}EcBI|DD<*?1rF`6MeHxy-K$oFr z8@cX*nV&oudOi9*Sq{sPXe0NL3NXN9d`HTV+Kt0Hpet$cBzqTBQMBgZ&h3u`9|0i< znv#eEENQ*PT(R)><<^&57x9`NBcGTvaYnfwelOlnF8<8q@K)Qz+%k`{T{}T({HY=2 zNkFvh2@}|6;%h9ca-o7X&3A+uHFVBVvUr4nl)bsr=Cy_sov}8)X)ZIv#ls z$B!3eP@p4souOjxs38Y-%+*&4e`jfeZi1lqbx4{PfR5T12|7W2lK><&)Gd zST0{S_1=}fo~d$0+SeAPTcWmOrVqvz#;C#R5pc56FIs2-8}`JVaEpocnrGP6hzFk8 z7szg!3Zq9r!@CztKQkuCC|5vt2tRb6`z>J5eJpbOuG_YU-_lxNVBjD7xxoLapXq&q z`sw7Q>6bg~)_^nApw8|AS!xCMDP0%eBYl7ik+$=xOBAn#dTq5TAvZU-ugIG>Z=Uv{ z?3|S$&b$WjBztjc8#^P14|e46bcBA0x2J%_cfQbUemz8;8BPUPR_j)l64l!Sx;$DS zB>enFPkFcY0&WvH6G%^4l<>DrM))H34?xOlFP0zkR|@F|A(O9ktxUl6?zJQMZY7ts zvV}Wqf%7=+$H1g&%z-E#Rao%0Jk_+L+ero+MaultRkGv)xy*OBkFdfLE* zrCQ*{#K@=*4m7eI&~h4T-}!)<#a3Gdlh5`EYs0rcU zxbQ_Oj{ygpQ?@cs1`O3W#^@uDT)A}~ksF{7jNHpk{YN?X*Au9pLao7T?>`Mk9pyM0 zunc6@)p%xfZe7%;Ae}U0CGYnaBGK>C;aTqHj7c)*dy4~Jvt)|8*~s?|pEu^Miu3d;jab|MT7}zRHMu^S$fZPm4zAum&S0&k9mM z$>>`J+}A;}9>}?s0A|08b(iaCVC`R<>sT`BaH(gV7DtircqF=AhP)9Ryo(pH7pet-qa8xG6;Qg(KB3m12E ztxQbP`Qx-;yqX1G@R{|5goMJza>?L;OOSPuW?*Gqdn6nSm!IT{+T8~o#OuZPzKO_w ziG)BpMxhGab9n@d3|a_j_)ctnR5QhMeg)u0y1-g+P-SQmmoUKk#V1C{uz9|kJiotC zfajYJaDVf;0>$4C*dPGW{Hw>x%4pmIPHUPB3=9}IaZ3`@%wUe@4+bFBi%EV>kTvu= zkXiVJP=E+&z`7q$qPB-KCP7er6)qhg=>Gs*Ks$;LeB_!Wc_E);7^wE zjYm=xljsbI{&OX8ttZI@1*iNr1CM;*+q?O;C|d0tzR7n5ogx~eYh5}*+`PPKvEs<* zF(=lEdS~MGPNk(O0xgmdWFICr4A%lcBXZTW(nf#QIy_65KO~<*teSN6p zd~Unu&8;&qKeXD1aFuT_UOUyX{^b$*GtdvFdCcB8mM3~roeX@;3 zE*s}-j6b2Z{>~Mj7&a)^F0MBep+4(nb=OAA6yVtQl9G}VBSA$_CH>lD+t_s;L20%g z?b`6hiKoLl`e-0IV8Q^zfrX9f`5mx!N6gZrl_%r;ZFMCDK`Q8=sbM?^--@#h+!zV^v zg_19w>PuS@xKGNLHc)$2&J?56_PSXuu0lrVHll`Ad&PPhjymd~Q-}+#rwLlKgy*5o zw^@q!Ap1YPCd}a?B-we6$WXc~*8F23Lep(@OP95p??Ol4i{cw8-=@cjqbovLdpZ}ao|@{7Xqp4;$687p4u%ZO%X}qk2fFEsbjBFvmM44{-R19q>}W=f>ZI=(bFmC!~4LhRZo z6~_42C9j!kGGmOaufnBKEFTzO`FEvnsqj>6OCI_Q zB5OmL@po7$y3;HuUgAu7ApL+RDFEy>gH!V&JDkwM$WC7^6-qnBL%6oNIh+E&X1t5$ zl&_5v?ajQ*tpWIy?l9%9(a7<^mcekL$>vr9tg@hP5M$iIFNE(9vL*iPV#Ao*(=@%p3pF6YP;v_t>!!C31cl?{{(LuB{Abiy z0gnc*Q@-h;8F?GR01QvO6p$_e|LL9rtev*I@?geyX)3u5uj|_vTQpdcEB|O*4sJ4Z0wgNNF%J!2;e0rA zC#kC;AQv5eczEc2Uq)IQ(sI_0F3ASX91pFp+Dk|_nbDZnK4t4xDB!F@Ycft^1Pb(? z?W;KYp$mZoK0Sne<>{9Y98?Apk2eN(65*|gV*j9yHq2z|c=K8vNrN^4Ryne94i8f) z=bA+^TXg<>8QHEq{bmNC@itQVNj`yW%qoN!HrYePaphJB66faTx(5fZT9b@gR@%mi z(6gBQZQFirHCO3Lec<1t=q9r?zyS76shQ!(`hh~0&{rL*Cn%Z?Gjd&fD0T(TzGZ4F z$E~AlhR2jP^_Mr7GI9=!{*n2iBoH71X)s1#T+zlRgt3b*dG;KqRxpq%DM)<6;ExSa zvh~)-6?~NMx*lv=l~#iwurMfDkBW=Q!|U*z_ke0JP7QM%KzXLa(Iei;S4YppKbBKP zn^6))v}D~pK6lfG@Y1?%=@|C$FyUpSf4gn8T~9rm#tPRG08x}Sl-c~`XnUg{LpqMH z&5N8$nnfFFF-0R>c4{tA=ro#lFzSWVBM?QMlrhDot|rCgglw8e@m@hnuy?CoDmGUQ z9;NliQn&T7tn~N)fVZln{5$d)#B$E43BGSfHBmT)ZIXV(x{KX?XB358;G~Uv(nq@A zyRKe~o7Zx+8xO&pn{W*@ZYB;lH_$LV<8&P>&gT;a{UoO2+ z9h+6EizDGGeK`>~LO75h#Yo6MV<9Ld)_!zGO0z8PSmig`t>e6XQr_KCc}6%l8H|w| z?ubTt{Me+pYy1BP}>I<&?|8Tpj*n1XY4>&X{?5(PE2Zx3{<;)R^D!@|N)!W7{;joRD0Moi}9 zz8x+s*nw@L(x7@j+U(eAOHd6(vut{eM=HO~qjF@lvtm;^G{tE90z#ND0P${2-FpuY zgT%_woz0k%U0NYq`;VhDMEH=WYT4*}3^JvJaj^G^0yy~IOmOuGt~0DnV@z8wyd0Y| zM)h2M#e)Q|O~P{pD@n!hoKEXPg4g)&xaUR#v5s}+=i0O#wr5}g7f&g{YJh+CdE#1+ z2o)MGCA{dnaU)k7l)KbHRhi%+((yx#^r6(x#DRaO|^*p_e3RCNVlDQ?31LI8< z`FamO$4S2F0PCN1D!PU!OW752QRaRz2Ct{{ZkD`4eK~jK91u) z7Jq|Ds=FsJ|j?>May|Ej5+T*2~undYF-P z?6j6WDjnFY))sck48sqgP9F2f_c)&#MI7WEcOV zA1p$LmfN^+?QqvOFBN$<$r)PRKVe{pck57J1K||@Kk^?CkB@`9%}sERz5l?9!eH{EbCYnL;K zbIIoDoG%*4R5B8d8DM!!Z#imCsl7a9X5vvi?x)kAmt9jZGiCkG>;CE3LMltk7t+R<8NPkTt`xnJa~6;KhI-dU zZYHVZ#jN+r`+Sv{`a$ffI zT=4Lypz=8yt2vrzU2fACS|4i{t^#+iPc^u%azMwY`;VOMMAt`cM5$VbO7GC*Arv7J zlpTfK&^3TSIuuh+pZpDi&&2pp!EDQ$Aa8&y@gSO83T$Rg*-2<2+VV5RKQ<^p>CH$Z z`7A+h^8()O<_pMRBX!mE=UD&<(S3u}7Av4gO4c6uP0+kN?5?u1vX8xpr9>C zYH6u0u*S)KHEXE3{A7j-76eY`ALNx^J!5vd*FY1{mEqZ+scRbuy#L8%YaZ2JtiUAo zK?3DN`OoqoCH@QpY2AL7@GlXuN5RCxN+~ZFOyp$Qa=!J`!>0}kfOOcC6X>1A2cQ^_ zKBy?3qndZ8LloxjDMvx&s1A>s5WeG$%BXEC%jV>BTxoG}@lB^#N`#023s-{rGsCcdKIr)^mG#O% zkjy>O5jiJ-1b?wyF?RMJ?+>Dca2a)kfyqPOpJQ#lkc31=5H`VJ!aN6yO= zbdfm2`(2`J``p&$g(+<9B0fXJGH2#MObexO15L{Z$L( zD%OU5Y5yYBzrO+(3FHQ@M81~)fsS8z{P&Z!uu3aLUi^5bq!ko&e@Gh`7}QcbL_b9& z8W3PZJ5XC%TT}A#I3}q}VsgkK56`UNucb{>PkxVL!CCD>`5*iN-poOGlbJbeX=%xu zx3Fzy$Nd`3q>gYHAoBsjj#^rZhMgh)MowF`sGD0`a7RPexu~e<9INFK{KOgk{{=V`3winSrGl0gMVG+e8d4TH?*Dg20iDe=N(Qojg77!v!C3gGfb3uFVI{~{?c*fL!!=I}o zsIs!s+ZOZZ&p)K=DN`*$*KgJS*ROwmrM86ETJFn<|9`F9Tg)89BaQu$|CWZogboQ> ziUCrzWIWjaUxJJT0cC{7VpmgmU<_QN15|qX*qfkP5t)L8i3#iGhu=?5tP*o^DFYej zVm9veKQaQp)ES=dbMHE%V*WyL{Qtd_`p1d6xeq^OWLy~T;tXuW|79P(-2@2x=Bz}5 z|G$;t*<i=*OAs~Gl^IK4Q0KAE|>*2z2ka;l!=Xv zII@-S*V=xf0}zm0q@bY~;|IEbBNqv3PuSGi840(YQ+@sY(%RbEzeKWL;U~_b|8KzA z6gs?-iII^JA|D0XRl*`7(Eil?xr&J(x+Nteb0S)Co^#v^7e~T8O9KBwBRL8jjqfRb zbih-apn}1}*T50GSc?;27Ks69nK`E-e>ex=SQ7#wB54N)2eF0!3k<_AGYORB|CurU z)7kv3g1XgpfihH3Q6Xmk!qveqzPY&xX#qaNIl=PH0u`$NQ{5AUOwMOY`on+z^%&|R z6mCDK)8m6U$MWiGk;PNc$>rBWa_F@1;r`o@v;Nv}j{sxIU z3sqj^Kp%8XO3;wbfKITXZ!R^bJ2^sQo$}|c0!W+Rff4ZKAM*b^E8>rp-<<^g^pRD& z!%RybzOGF4$P2M7?I>J3uYtVLNotP?5j><{SqoYGmp@mJt*0r zuN>au@z}?sooB$eb?%L6+~1pux)_1WhyAnQ@du8n!)igZ;Mkmc?fZ^Z)$E}faJD%; z(&*(^py3SA_tE8_E=~FWA?Jz_pmJmJtCJ1Yd<{vaJckqNQqW}dS?lJnmXco>5rrD- zBZruHn69*3I|Ax0y3WgxEiS z<&^WjG$~s#?N9uR3(kH5M-jM&pvvxAy;zJzboAgw0GOWG*uV=3-eS?1LCCSG#1T*a zwqJM_(0Dk6WUCpsJ$~u~gdm7n1L&StJUk0d|CEi}C^XFw`XfGcx_gK_I4)CP;HS@g zjJPuBDMTMBZ4mWC#kr?}`{@*@LlBE{X)r#wP5gf^iW2ihD11@4c>-red1e4p-So6B z9AQvyLhxQWHs-p235GIc3bCk+46&+nVEpez#i85Zz+*@ zKLh=weBNpI4?-oe=7N!V=^sC$e{||jmQqkxC;ihjdHWh7KZxPYQpC*n5}JE*;~e*& ze{MP*yWW>eQDDfJSmZ)S91}PeANOY3Pili=no?o`9A^hihzMEo1t99{hiP(%_3Pv7 zVo&N<&}_piM}IO9l;HBBISaT=ah9eVaqZ6wB!xhLY~mzEf=FrqNz{$Y{HOaf24F7- zrm<{+*8`!|!vAo>*nsxf{Zjb17b!N3iP8#Z2@5v93x96>h7!PoKnU^czp9?V>CA*s z6nHnE%JnndgV*$aLMTS92r4c{E#Zc-=O_nYt(UA7g_G#&HAR+lIPH={cH#@1VLy)r z_!Fr=ddHz^Dk(J#*W7-2^SsiVz5y2F*k4Vg3b*d?sJ^rBF ztM(|`fJ_>jXdl{)LDyAl;y0iI&u=F^Oc@aCIo-yJGo+XACElHTma6(m7~dG+C6+$w z`HOSJK4BT7S<0NCnWSd~MW}CmJnEqP7QJxjK;KpI6>}|h_NoSH=BZoz)$&ntHG1k% zl}uf0?ex;kmij0wN|4dZe8!qFiS!pP2jSj!919P;x`m$X`_o1wmVl%HvcXy57-%&^ z^nG>N#1hLQP{$eHP<;ZaHj_)B)z#Y$v7|D{MiCfjfyAJS)uzYBBr`Z={U!h25+dax$pzOUj<-p` zh7>@CyWqjJPiN@ZfNU%BOucxcBkP|29R_yaNi18EF;R!5aw}IeRUht{aL}Vo-O%NP zj#bf1DZ`#$&zSfOczAcD`A32C&qsm++9}#!)K*Zf)zG;1)=Y;NIx&!v;;Drh_#)fq zdgA9vkSEOLKU12ZqU7Ex^GaOEHK86&JXxEfT&nqMY@O@d(M81RPRhn}Qo z0pu^nT+H^)DN`ryKd!qKPpG43;Phao^wJgUy&(zPSFto|9=mOckGLKqu;>M-b4pr? zAQfmNhSWy6&XMq))0jat5YqiDot;wdOwarE8lf~*sWGn!tO?(Sj!ub2W>$yd4`C0V z(b^k`JxXE^9fH(gcQVqC-O0ZogCpW`?Z7n8Q1t_ux)zsdvvC|u$P+A3ePwX@&_Sp& z^)a_|4$&k9?S&`Aq6gk;BP9E-3J53dh137maF~#G4bjNq>rRp~?JYNZ?-NF`Oopx8 zz9BX4tK(Z27db@W4?k+H3EuTeVjWC^D3*yu0yeCmn?~pos6DZ2srCEEj~}C4vip~b z2F(qimt{vsln&dl#VGUwdBv-b0y8MFE-9e(gVBa~J>Evos1{Ccda&M(Wj=X@m9ZV&41l$ z8#P3a^V!)sG~J+#YVE46a{tWTi1+5ItP}V<0*5YQIhzeM?H(-PAHNl0;W@g=X>?#F@G5v@A?Q4heCSr)sZW`IEQx;q8yI)rbB+Q|f29kW3U?-mYN5SV5Mf zr3Rv@Ws*LD{^RH|gxmp_gcop7&%6RB-Z@PM!mMaMk9X=6sU;?iT{LnrqVhd?9L#@S z({vXuWzKyh=R0?UbdeKs?x`;^UKz!TmH@t7iX#ufkC&b4II&@P_jQa+v)DO`K?$yW zcabylVCa2P9^DTyWYPVd1xL6wT0mu_wTA1HpMM%AXxC7iZeo;wPubIFPh>s2E3+`g zaqiwGc4VhlZzHD=A?&ih;W(!~2eWf%EJrPj5`F@h~`sE1q9Fo)JKa4O*H2{4VG|Cb{z@ zloR;6>FMbh6;s$`y!Nggz3GY=r;d#EKO#&);->LAV@PcrejXnVM84D7hWPDUaT)HP z^@+2!)=Pk6V`gS#nGjzwt}G7ZB)|`w=ku0sAk!K zrejo=Tu@IQ0?kC8h|Kg<3{AXYf4)^2G@l|*E~+UYQd(jmnDsPpua&N9&>j&nQ_rDj z#$N-PM!LsJqGcxzC#~2!!`go~@WjCs_t`i?GyX`hAVF@!Gc%|RY3K+y3|e?)u+-*5 zk+v0`8M4z0u;Bnuszi?izB>7G>6t5Wx`EwoafC)IgXD-RxA#na0-}gQd`%um6 z3eQ5sMuH#`$ar+P!dAmh_tW?xJl||@aobn%jAgVP1U_|yW1r3p1GNbpq-dGj5(tpM z<`Fb&$c70*Rf0yZuTAj4qk6!#k{v-*F$T~wgZ9cA??L~;C>45C zw~mO)V57^&Z?QSfxprb#FhG2Km-u-$0`c|Qi8s`0^|*oi`;~8}vkvZ$ z-qRDu8P=$}V;VVnef(3WI3*}f3KCk+ORse$s7BS`7&&%{dx+$K9Z5t^8*8`s6hRv< zP%Y{Z5Eb|DB$XgAk#~7=p*PPaMHwL+bcBmiaJ)#lKKe$4uq~#T`_U<`pI>(8?uG}Q zfQF5*f2!7a%`QyP$%m4(y#j^BHiFKflt!7ar(WTpdo@OXbMo^<#Fs3Eg74BMntSRc z8S`R4#Py6bsE1Gi5_g&I!Vp+WV#Pze@=?vD$7cXeL8|A~(_N2LWv-dkJ_+HQzXKdI zpU3c?lE(+x+?mH2!0W|!As6D*tc{`_svx(>_k2KTWC@e!h-MTG8g3}4A1GDJZ%vgt z9gK>jJj)t^gJ(gA&XbK_g$TWTv%H0h`xJ%FlqNA}ob}m_QIg_zF5A;-`FM$9V?PZPa5f z?-N{W!c{}mXTYD_Cx&_Lu3ZrM^weVVd&SW9ojX`nPRYbrMBm@Mp>cd9`jCL>DeBI! z8|uA#4cd8rZ?yBCJ$pkfm$IiVj)8H_!>J|;zoXeQUyApMc=7Rq6RulJtN>SMGwRfs z)0gFHE>W#br{j;s>kUy=LuM56uX4Rvq?4BR6m>|iG6y@j`n4;cGODmjWY1chYTEL7 zF;6#bcwf6P`26_=Kw9dLfS@=_0}=}e**~TeL(5H|HaY7*Dn(&L2wOgQu@}FAC`xt8+rM_b*ow}o=f*UnWKOOKA%d5Fen54J0kH=ss80|!|6{&4 z8(fH#qaw-9)YDCs#LP>RA$&uML{ZhVGN2Rrspgv%CGvX{pZgnW=W3voxbn}NAckRD18pMA6e0$;Z4x`w9Ab}xs@INOq z_2D%+{==~%Pa&r2Ms81qc#7;)^3v=T6ULMxtg@tq$Ab&BvQST+&#oV*ODlSZB5N{O zGQw>NG7Hd~Vl|Y4i2h*f7bxyWn5&nUE(M-BS(B1jVUZx$wjS(BcI!vGz79&PLg^lx zUco*3kGkJ&u7nTkB=tsr=5>=T!yYsjEGT@N*T7v-`*FEx=$h-;6N!0ghTa6%7Gnwd zvQ_Ji?Ic5{X&I{_?^^z+gC;OpdRoR9m6lwi&o zA+5n|9kCW)=DQLYsm&>c?3^-OpOBI(M^)^W(qFlp!+Sz9u-|2)NcbYh=s?Lq!pN9- zP2*$nf$7rP%W1}2s(q=YKX+A}Ngz5IfUj;|iRX(Pn9ZS`{;9On(VvQpga}Yyg}xE72oBfrQRXxB^HosGd3$eLS>8qft3fF!z4+2h zkJ`t*orshxzIOFuv)J>xyDoWb)>C4*Xkr27A1?=8ck+`hPBKuT1^KtxgjX;73JT1n}aR_V?GhDKCW zN?IC0y1N+^rMo+%8JZbt80NhPkLQU0bFSyTuIKCX1)z-bk?^pf7MVXFO_^Y zVwf;~w+U2HKB5-MkW*~M%Vlf7soy=HONXw~h=HeB#~uKpm>bwZHHRPDri`CS1ZM&PZ2qyJhxduKJHeyv#0x3tAcaLq!7fMO5|GR8mKsFV(<#&z9D zKb5#wl|IB@sfn)pA4ro^|1^bu@iXu!Am-r{ElLKM9EEsrc-Q#eqO?YS+vjA=PrXQb z`oCvnaYlSLU62j0whz16?j3gM8~ImZ%un0%C&4k7pXm8aNGj9mvyIOiE%6(z2m9fA zx8wW(@D~A=>>`@_7{%_+H8>FePlx=eHM?xj33T>+_EAkcwiyklQ!te#4iN*|<*_cT zjYP9n9p>^xGDfQC4CLfR{aOTyQO<9>Nw)&{$Gnr^*o7a)>&Hy>0U z&sVQQs5|7`+1K$a4WWi9#2}rm1R`gTR7DHER0+^=J|t+)1gOG<>!5~jXQJ?22_E?$ zrFWt;m_Syo%cBs$WyHtp8$60*!I#5`orT)70`1Em|AFrN8K(GqSqWfS^?^!2g+0=E zhW_&M0~H03@9~|v(xRc&aQIqIf=19bF!5|C=+~L~FlS!U>$!$YlKI^Ck$-rZx8(uV zmMbm^(~Xuk<6FxwF75rYCC|RM22CKAQ2#_3xbGh6i<(X-V=iaWD=l4f`=3Q2WIvk= z&b(pF7oJ@O1VYX>pdJv2GNSo~2a^_MIQ_st7_}#~1;9Dd8Zytev;Vw(fD^v~)Tb8- z<{6K1aw}RxfQt$@3=7JL*EgQb`Aaf>(d6`3xER!}8HgtQI>1p{$qlpFi;O)0Q`64W z_-IGz#&dWX?C1r?{_AgPlQBVWM+~JYf!4(soLCxs08)Yto4`Qsd@k9eKhqOGX+($v z&_rx2Y#9iCQj%l;fJ!yRWB^SWn}Cmlw2?TI_4}K1NtghdbSPpjUOQr zj)I8~ajE7b@1IZp=i8@$5xN2kVWZzK(*g@IV@Hh%s`Lp04B$a(w@ndVpf>P(zVDy+ z29OTd=TI5*2;%b}%$iS$7>l(%E$UA9KMBjzpZPXmH1fgbTcCcn;mS2yT#s*=kUz3b zXNayA01(~cc^UwSNhpcUwf=HDuh$g!k&aJ6+N(XJvD3d&Tg@N%(W+9~e-P-m!zbed zLAGHmP{`S|;lB|n`!T>z;LN_e|1Pb6GZrDO(m9+^?kzcLOpo1fAN%-Bo^DTRUeOx)n8l{QnqS)G zN$m0T$|Z)|UDQgchbP@a#oX-O%c{F?iX4uLz~{2Pold>NzXgO47kHz-TQ`>fP%fg8 zc!@~Y5;nbP;|T-SFB_^@t}91=4od-+xu(MRvxB6tTo+TEATwo%Cd9`~LMVYxMf$=M z=ln)90|KqMfsnEz6;*+;o=KyvZ3GSn*74%-_XTr(8m=N)Y zY2u%mv403pDC4V>sJC#X<#>KO|2Dq@{y$(gr`pJF9%FMr920-W3;re~$qWuSJar*+ z=)5FnsyeJirGZ+|w@pjI905xHEQm@S_U13mFbC zy2o~!DLfm^TLRAlTmT!~I=JO1T@U(mYiG6P>HBpBDlgVnz5viszrSuQ9N;a!Ip*?C zy8XOjd>$HI!Il;ls!oeWdWOiw)k&4FwM9;@;Dv=@WZejL>wG@h78nll9Psg&4 z?i|APnzoOu`KgzZTuuG#Co=T|b@@LRfjcK!TUa6rOwwEXz*@yd)=^f|;fz$;^wQBh z=le1Y*~vW~2^EdjDvln*krdx&?a1XXiIjvsv?XC}A2Mxqq_K{A&H8|Wv6~qR-~em! zzveQQj6IaDTKQoTc6}3-bjgHc!t^G)emH~dFipfazM4`CN>)$Ma9HT$SZRlpePl-y zT0fGR234twm`h$8i{>P7r0h$!DYc+e;5bsuj`z~ldsn)f0PYXKw9@{~kfO<4I1A{Y zTrPIy|EX&sx$+Sz__)ctu=S>Vp!T^+BdmjcXut7+pDe>}a--ol^OeJ5H^=hh>U(-Qo)L>z37&d2JlCVBQ9H3c*>}l~-7^Kz7W#_o?MxrAhdH+hPnsm9q%)$mW9IkolFkVO-n49UI=+gQIURQv{94%ad~34P?R6ET*`P5j3}+OE=` zt7IeZIvdL-#7*$&YFT&fO(NaXAu!F+PS3tL`0{HCb0m0c2WDC=yQJ|>(h=0c?>Qx0 zwfBzH;)$`pvTa{Pp2MS%F^AP&u<>~5wIH_*t-^C5`E;8?t`YXaDSW{!af79OA+=Rg zt0orfW{~xbU+jwNk_fSvCQ2UGYYyKUt#w;7G@GqNKWe&YnGXyR;SMToY;mhM?b9AD zr5j$}&AX=2zS2Atd6Yd}-JunByM)a2Ch~|3bxh&U6CNyiZ`?#q6*X15ZgJ5v5JXYF z)RGcOFu$A{mghe5hqIRX6u3LSOAh=d4j928yn`(iX$Ovi4jeU@tn>=E5}m9%g6_G$ ziQR#~B|7KZ4FZPxCsWcbx>T{3ZbmXt*j7+fNLCtejtz==SoK0m^iYrjqV!nbRqgZ6 zhxRWN&9jDDxC8V}T?X*v(-iWgt4^{ao6M>1p=OW?4l&xWVV%(uXaixT*O3vgA^S61 zaBaeP?E|T)W4Ac4jN+z8cVu4fT;o z9KvOvYPVW%^jdDMrVKB!1F*OSBJP&RCwSfnAsTdPdz;(Pmu2^St1QK6mZf+#1Iddv z^zH*rz17Ryew|z%N_6B7FFfZ8%iBcDtGb_iW`?Qa1-Wc;63g+pxG5L zcIGfZkc};QxY=_hIGA*AYg3&-T+v@Ql54rLn(MG81z`=2f`JwD@1JA3*shb%tzah4wy-X9!;f7dD5w&Xt9aXwC)2|G(e;-eC z=+0BO+=)#y48WBM+&$?&@7B>Bvhqv=aj=#-M6{>DRZL{G9z{v#gQ|C!OwhkBxTg{Q zGyn3>UP&*wqA7H~?<=1z@c|58pKEX5m$n`nxumnN0~ zs*V`?m}jgoUF$16-OB(*~P#(U~0+Zp>-t@_?&gXR4H{&*hsSx2J=$QYDuI$j35D zx{N2k%Zj6bp25`%>W=n(R4}N+ldZkQ(`l`@@CIyHTdvpJ8ZYP{@4W=&+G>*68PgmO zG%xL(v1K3hP7o71S#(ax`1sRP{^t%?(`(qZeXDNkFL1^U$4D-#FNgKjg~%GvyY_{! z$hcw&)AExlS7^t_Bn}W(Zb0IlqeW8ZYmz;wG&rkT&oM^$yuWo-Sf zQ738g;jTqFxzD|$oU=u|Ia=?<>Q-~)bfhP+%y@p$3m62|OJLhgE{*$FkR+gxjq!Cu zo|pCepV-_Qn*cda+rMi^eenF_)_Xz}A>yybK#>aibDN*oK18ZAT_Ql;MJUD@TWxyP z)D76B-k}?~-pl^k+`_(TqlthU_^v;q_=cGzf&E7&Q8Fa0< zYk&2l(K^yHBd6^dPk?@nOK;l`pAv|^w*p{)nPjs~8^bQ&(#g;`-?RYLq#~graKY{?SfwRW{r0P5I<;MSNO(H@*az zm?jI^?@Jk9tFYC- zPRRp_(P7(O}Z;ri@LTMj>RKbi`H4Quab@R&(AjPyPG8D9mw$;z2|HQ)|7_XQ={6&;Qo2%W3R zkPg)L)-;7GoOt<5DLIQ4q=;tnSEt*+4)axD1eeK?Ax`EA&vIY8H*ZBrmdPqw(t;XY zPj*tQ#x`{D`^&V`Zsv~W`b|gB#gd#IatiBcseD^H zP2u8+O3&|2eq46&JN(GcM|~Np?4xNzcwXLP#Jx+MJDJ%u-b`EWYH_$_G`B~qB+!?* z7Y_#|(TZ!KckEfr>fB_@J>YW+S`~v2RF9S*$j=oP=+BAFX)o6n9S{ESNVkBhs=s^; zl0BU93Y(~Rb2O2Mj;8IEWYb|0W8kk-;^&O0S|VUly0ke_oLN1OO>v>{wYg%G1qX8U zjsUe5=xA}xPs^c;*)z{M@#9wP#j2UBu#Z%Ki6o(wZycwM39b+HfV*=!v+KPlx`wJr zqj(9liwJM7k*nWlTEu}jLcI3^rreyRuRg%aec(9_NiLb#fA5kE$XV)$9dY1xN`!{11J3jm-Y?jQywTr z$ZUFbmG%%V?)7Q{)^y0~m?}s8vBgSHR)DyKRE~LhMH?IUR;v8j;O(QT z#U7-y0?(N1&SX$A#I)C2VA;~9M1VpaaH%7+=4vRu{2&7cZ;k`g+rLqI3(V3Z{<1;lx#XkbhAxe zo+)zgovS?VkaXCL%nR&TG0cx=)Z!=JXIcBhP(NUL=9u-vP%j%3y}Zp{=pmb)>1x&z z3F8o$HY8epozqVbN_WlUacppR&S>pOf`;2$yq}AYx7sa*^DnwWmP$5r=j;GyR66=v zby?{fFDGdD{NR55Jsjvz>tg{lM=U2DCpp_8L;{_e<=5yKZG*TsJo>E}Y;H371iw(? zi^6%|sH|-un!JyAQfLQw?zivEuyjNPPU91x%;03JL$>Yr)1Ob33n*TlTkjK>-<&8i zeo6lwx##YxU#}|vDW37Z1rYCU}psPjP@(dD2YrNcpN9e5j6efquwc?k} z(>sgNqP|56h5B<(LyFvM=*t9UMmZKe)lq>q4&~omD)X`{9DhP-V4bIDV35(!=#{&= zG1<3t^{U`-k_)s;_~(80iGF5$TJo{^x4_fexdG@^ILdOEMeE(JQ*UtTKt#8W4YCHh zT{wx&OZPi@d_?X3av1q_c7A$y$?f89?We{%d~a~tM%__c&vUwZ&Q$4NSO5!t6$bY= zm|MD|N!xEwED7D3{dG${dL&A~0G?5`Rc8lnjn_VpSF9M8308s4y5%#j^#=_q$q-n% zgL93oBF%+axKh3%Rl-u{1HBL|{cPYf0)+|ckzg%KofbYdJWf`RbKYq=h@MAx@qY)Q z+jbRYrWNbUf+t;^(qW3z@{wt1jtB-cTLfwa`B`ey@Uv->;X>s##i+ZT+LfC7)lf&I z#^FMSd9ip|ky=^k#|XB%BWlai-p$WwX^%%M=GPFQYFp^h%fTsQ&7iQ)+RR*tlFRc+ zuf4NQ*i=-nf)+Augwh;`sU6x@Ol2CIidtZI7uTF z^wQ>(^%h>8oEtQ)2&=f+3`|vz1oks95)u#+og+zVTrL-P6eEW^eoWUhT_x)RIZWuz zLjqMR$J?;(orGJBQSvYeQ-9d`$G09@(XWH38J6@P)_(OV&qwdN7__GxItC^M!9Cs6 zbh?H69;0;QMc`bmUHnYp)A2?^jxC4L(g7OQxrX}*P?K)1`d7)`^*s3obnEIg@NUiN zIfW1TLPwRF#x!S)ynp;jLU-(RA7Ug8HWrC!!*#-a=45$-UB`pNj5g?nr5JxM!>K1S z);%qtApsp;+lV6Gzz++uC)<3s)Zdz#TCa82-337RgCW@iOMx`dNTe-PjM&H|qB>o1 zncK<03;Do&tTwF~A!J@NKVQF|=V+jq#8^sfO`Rc6!%c`YMvqt3x&PEG%uPu#VX6)CoR^O#6K2gzBC4<3WcSH`B36HaDR; z#rgzuN*fK|WlR4g#_@71E>3sU05)Qqvu=+28pseW+z4|a72ngqzX*hvGVK#lNk3-R zKf7eKykksSRTkpql}dD-AwFE~Z>3(Eyv$LvieH7QZyt>JAOX{^F}U#B6QJqqik6!| z3Wd#ADF&xHciUdL5*NT?sGK6J!<3%1T?6K$?Ku5(X-GJ5J{=aM&F<{o#8f3I1>u+` z=v;sJkfzAEAK!8K;8+UJ-=A2xmF_@CQ=TsG!Q0N1bFPMxdK|X3$GMQUx!tmNbMr%% z{#suQ;yUlL>r{|V25Su!XfZ@5dKO2-zZtEy@tSf$GO->|T)W0A61r_SG&vTXwlR^g+WJ$E^Ag8^j zijWS|3vn12I2&b8<7t%tzQkiz5{Q9gjiOs3`&-B2c$OYzY|S3 ztEdtx9zHMc&@Gg#KQ#6|-rmF~e>jg*QY}Zh|Ca5<-4~qe0*`0{4F}aa8A*q zk>F3g!ClgWU!Li~OZ(7u3l<@?nUJKztPBDLX}Pc^1GLhKc9CBF5hTAut5||MX)Bww za-jdJqU(-WE9isy)MCHsq#I&kwbxK^brma(xPutzbo z7M3M16W<_F75Y4vWjz;-OxqbZ>z>CCt4dDu?dd~I`f7$o;1NCMHO@DO!S}q@>Oxw( zSNW&WCm|l&b4japHrMM;inQR@yBW@P71Z5RaH1V$1H~Qgd-L2BA@k-KWux)np8WV8 zIW4i<$)Yb};5loqm6zT%#K50c-rHT(Ufvch(isZ;zm;lVjDHx zVS1O}jg}*g>l8LjS)S&UBq*-+YIE>g`gk0D(Mfxm3i9LO;Lta|)?7WEpKi?oDkz8% zlc$0w@}1nzN$h&$wX&|525J8Rc0e4pb=U9I$8(vzRo?_eGA)<;Jy_CNL7{y48B)ps zeaB5M)^=1zH(N?(64LlNKL# zyT!uV3#jv^nj19|E>C%>!>fCKX3Wuq%bfB+ezd_G=h(Z=JL+tA8!6Del0WkH7Vu_7 zpAym2)4K%D8yi;CDU_Q&H%aTI{!k^B0IG!8~Ry(&R zT-HWz2p}jPMg(xyw1E&qpU)3@wNfQe<@g0mz1P<%-|t1`%1(Q17nK>rPzensEwVir z8e^}zTQnMc(p+aOdlJ>WK}Bb}x5?~A1j~+?R==k~kkjeVzrG)(J*{cbkG&FS3!Tor z%MTcD1`|X3WgL`a<|CC^6T4fk1=f>@o_?;wfTY0n@){auNREB2%f3y>?h(YK*OP;d z>E31I`AnTW&wVN?HQ}X{=NlPMZqt zoc7kv+{5FjgR8{cd;aq1hf>$`@^9;euPPcsL{6BRsa)a#7I~(gHACW`AjwIUW4QL2KdQB%>uGd>ZZsSznDDg48+NX^Bu&s|b|w6!FXNWiQC) zB7X+8`2>9v1YFuiiVTe>D(xMX5?tCcl~V`UZrJ+|Gr7=_NaBFDkO1mnD}X7`Xf|;1 zOj->GQ(KT+d;SPh?vMMiA#t)s(9OuC)@G`(tF3E*)TXX{O->;Mf510JqPB}2tJZdt z8kh*b4ewzu`i4FOBHH5e{)VE#yV}Rib_uOYM{+8|X#Ucf)1uW0|r%eKx=Dvfh zQ4psu7qK?SH4!VY4lKQDnz60rc#yR86BfW8sMK(H-@JX|=`Ix94(hR7T;=xaTo*NY zGo0nPLp*aI z#CG56k+rEmQt9nCg83Ble=i}fPVyQ$Yy!1RZ455KwVO=d1qa^;4Cxy;ZYYxK6+cq< z7H*?(X*IPwt)Nii$9_OEk*&^}{wAN2s;p7GA$}FolPqS0debUk`thH*Ga+s1OEDxMfb4i zjg^^nOI$gfZ%E|WpYOxKEnbzWL ziY0eAzhsP41laO@I#(Cy12d>!OVMTB(mX3&SbuBSD)lJX+QN!7VqGU7W7cN0S}&$7 zH8sGmbR)8OP`jRvlfOOIrb@%Xn%utAjNhw4)O;|#?prhC%|S$LQ@04>12S*OEBtZ) zE7-MhIFc!2ZKS-cBd2q)+dl@+<2#tQ{d;4;urJ5%UGQyC`f;)&WZ&Vm*{y*WpjgL1 z4=`JZ6^_?(a0%toLHL`81=KU(G<-Q2TE;ZhD0Skf3cD%pWo(utI<&sE9H{o*ZYm;f zSoKUG&yh_ex)B|%ZlT7qE!#e1o%f)|W&Peza}W#NJ#+QkXQm6q-yQbwqNB7d4?iL7 z!FW_m1D1zJDr4Eq;d5$R{T;9L>gQ<;Y#yXX^PFoqIte3hA>~f-%Qw*~v3R{q>+m9M zUv8#$LRYJ(w9>$Ba-uDY!JD)2CG==Yb2{kFvyl3l( z0cEK(R9{JnHp%-S7@Q4wkS|I`bLV!P;R9V#V3@hO)bl^lmX>0u8{h zuiy)i`l!WnMXTb+$V-cP)z>1A#9;xrlH=;l1npc2_cD8mE z%e;h7_Kg=Z#>%O`{p_`k#O)Bhr}|j)T}a$ZC7-NBzpb=h`z-OYNZH-KQy zvfGIohpqa=PH}Pd84vC->`V?PwFh(y-fip>l9+N^;J=(KTTod(Fz`Lqwx&6I7g-?E z9uYEC7TigdaNA?JQjCWi%Nd>C4K@#00NY*WU4&$4Fn`$N}+ zS0eiJO2Y37-YzkfuK@NaBo8l9oUo;SvYCo(1-v#3Dr9wPNdyqy9+J|BI+p?7$i`He z>eNSgE@1jo20wyT>iKu=B=zI#G;h|Oq+lY+{{7jcvc}&eOr`bp^VJLHyVB_LBY}u% zU_a>N$%v;Rg9vAV5jXih5RKma3Tf(o(?u_`A`(gCk|oK1+LxxpfQ|D-{l#25?N!<> z3USVW{NJnJd`1RDjG~32=|nLm*Uj^6(_zeAB*#?S)$Lu;BHnSzX}pjHL`4g`i!c-D zLb*}ZO17*~`uM|o5OVk99FpS83#y2CA0OB_CK$Og(Gh$hFg`7r>Il#BRU+@ z9xrheYse=E0v29MhCh$2Wtsl=KoWX~|J+FJ=BFB)D9$HFvqS3r6fmnn$)LC_=Az4O zQJoaVD0v1uP--XO)_B;9QUFQdaVb9N zf;|JC8B&CkIi?(vw|^h?jrw`?>s^y}#c5A#JSngSVrd$EdtLoanU+cCfwB!H|A1fw z&B;?X`Pj0;;Le2a7DQ$!JgkysZXbVQ8s>x=C*GH}vS>YeRS~q3u~tT$dSJKMV~!8I z9Ml{rV!Wj?!~z*{aeBv4Bs^j~K}|ql0QB9QMJXrX1k$_N#1=l4@Z(D5iPKJ;1<VVq&zO3zguk$6cKg+1}oGk@&qiV(WxFQy3Ve94dl7?sw1dG zJ6E;aC15;|sl3|Vz*pc>NmRlxSMh_B7?P*v$0%|L$)MzWdTe^c&o~%80P0=XW8$1< z-t^XlntAyy+Tl71qFosgML)mBjoK82emxx;`lo?#!9ju%Y#E&mGmk^u9wd>*d{ zaRGQXnV7!W7uB!6(L>foHI0&qDkIhRHb&X^C!F<9{S2LoY%C*0h7iBb#iVK(Jmu-s{+EzNr97b`ud z@!ozOk%Iwo{snl+MAv(w;-ec_L%b_;E4K9qBXcKV1Y;Z`BHeQdgL#?;CLFq!>Vu;b zM;CCWxZ=QR7totTy_+yO?z{@EC-^>K5)?yZWd4Ib)}#C}JRs{tkmT{HPt0?7)c1hj zdxM5+=f$+&9Taju8W;k4Bp9JpV$$k^ea<6a5xJl`>lacZT8Topd#+9GwO_y`aKoxZ z7FGc<`|Uu|rM&{qO+lq&23Fg-N$+G)?F51GYWQ@!PMaQ`wstdjk>h^5<@_LLz==){ zIGj)G;N(7cZ;+3-Nh?+%6bFheVh6$$sJ#jW-oib%(Vw=l=$(5%!+4^mW~(bPNJ_`r zWvuyu74Jxj=yM(BB&Bs1F-FuH$yMbdcV-7x)PW)sd~j4gvC68? zMiJvhCd9@4YoKw&_W6Rg=IP^A?t>gyFtuaI#hM+EVc^DoHdA`@>^6#^O;e;hw4{qd zyE;ERYuHTKdaG(@LD5kb&s*r$&g{%>QqlUOH~00E!au6DAB-MoqpT*-Bd;8E`h7PQ zPk2?ECQxtVg%1QH>n8gS$a@W4e^d=G_%bGvkq#ov=x*QdoqKAUFugNPaLHDvth9nV z4sp263BBg<#o#g_rt}C%v=F9ceah$>WA9&DzqG^qXrJ}!v+Sk!uPVpHQ z%RQo_(^^2K^ajgtDBJGXtbUJJ6~;5P0@1k+zGnz4cl5vi-7aTnz3t+~8j%XX)E3No z`LfG2!J=;FQd8_VH@(qC7M%+9xV@{JWseL_ z7PuD98NEaIP)~#ZaDxfv|MsBD)SnTt!1+g)`r2-@5(`UT>12M@7LPG3>D^~gEbRvmYXwwr@v~2x#Zt>W#xaD-I zK*b*S>U7sPQ|}o%k?7izl2WZ%r4KzC(gEM7>ngsotnp=|_PLtQPnYe%!XPEZb4Z15-YK>d4GvH0y&0 zssI}Fq`yX*C)ELFNb5`C`Sbje+#^;>Q0tX=DXmibQi zDw|d@hwPAbXVl`Q7ZM^lIyK__wxZeJnl%Im;oaWrk!VzrR8($fSc3a5eVIaO{0N-{ zP{zduR8_`>W4OJvBs_g~t212r

zzk2}8KX8a%=Zi0|I(6L}YDA6E0+a5n@qf*m;ls1yP=8qhgJZhWw+b3|*ke*oCIhb0 zPEC2IE~|SLRRCshG<*S>fQ z)#tsjF`C3VELil&ghOtBEK2C>UB$@}jeOw@nJgISgXj2ej`5C*0MJz^(p%xket6t! z6Bd8-qj$apFTc(Yfr-T_V?%Gm>Y&$o+>Ep|8PpL%wygX7Z=MX0R{;UplxmRLPn7xO zF}#!)DL~dxp}f2rp_+q=5d`u86gD+FcfAiwbnRB&s#qdd%Tf^9_4`7?V(AJ=Kvf{G ziUlTxEk;N3L{%tS9J)7gSJEUOa^gEY)LtKmv)i-OJ$<4O9iFcmO<)7?e(Nka8ivL&Qi1UBg5&l#{Y5)ILT2AmamgUmH;nVuU%dzccp zl%?LWXf4`(Xt@HW_PKkmh10I>W+Z!D_|n6TcR&Fk?qGW22LJ6Y4v59rJo$ya*31lz zCyQSmky?>fAs5OV#(KeN3Jwv7cw4V=c4)ORtlaE6tL;R;j2k#JZ79DiYmaK@E2GZ* z%06#f6P1Kk$51DO^M=a$=0u!rymi)C?;NI>kxw`?z1C}O9if509cN$*)Cy#@@S_6i zT{V~O+B6^Z=icB{R~1sMEKB$%NhYtC_aK1A!?Vr?oPcJ2@@ zgl4=U4x8;i)7C-A<@=+6j6kb-<#a{|po($*Mdhq=A6oPU&e+=3a=2mY3W0W@hzDME zVsI79ANpWtLs%HGRhNG4G$`N96sSb8$eX6j8IuZe4xKf9$OS|`i~`#dg@6+C!;M4g z6u-vmH4!DIklM1T5nk`A&Bz~FnW~mj?JnJji28)^pqYaFYh;GJ>4{!5s~U^!`Xb&y zol*BkXm{le8j9NnBIAvK$rTP_Ld`EbXNO$MLna;Cm6SnW7?u;M3FOZ05)1%CrH+h% z0vmD{Re_a0(3htnPYYyfpGl;vi`;Ypf&t`Tz8vhG#)*CvUCq(~jTIhK+;nI0D39-u z*Z@;$N|BP~QTX7LJXJ$=Nn5e!%8~`(;=qLNbh3hBcaI7q@=PxGy2w;*#x{wJhlL!T z!iLdpAdXlNzqLWVl}vj{z8=tN@GyUqf3g^|4HUX((^}at7DA?uJq#^+#~O|ffHFkc zUHqa&hLCRed?sRgPQ7S`{v}d**2!%9+>!S`1|(7T;k>PH*1u!hc&ZSDdV8l9bhz)E z8Bz%3zLpyQ;wDPV_jH;PbqYWAlj~^(F*&7XcawmbT<2U^5-TrtsY00rC8Lh>g4-nA?Q0N>lprQLpS7CvK8K3Jn({-v zQXA=l-EpAGlpmEYv2!p^F>uuAo?WU?JeWm0a+`z_m-7tVA;}w@Tx`lWoC&wjR@p>e zA@t|=Vx6-;Y0cW1y)1y~m1(l>_9&F$P+C$6Ydc}x1IieIBE5+3IktVHB@2;JEVM9v z4co3L6PCv=K}SxUN4k(S$m7l7%SL3{V#c0ms%yhHbs*L;!+WNrA8U4oJQIw!tkG{b zk~oMnWEZnU3#XiV$-fGXF}7}rT?$&sPfdNa>%MG4`;q%)&~Vq8);6U^AYaW}a7%Jw z`wY!RnFv2@>+I=xEJp2Jb!7S&6So;5ps$;nZq89tuirjgNB=#VifssC!7wtJlT+ps z@G_sk@%`|Gyjnl2ltdJI_~@H!w{wT&zVUSe`%cGtuj&wqf?WY*r|{0#k9!Wfnq|cH zug2qhvc;@-_Os7%?Y-u9_{x33$X2qxAmn_T+v#Tgb`ZVhzugs&B$JP!_HR1wkpr{H z|EbXh@{a{T%0a++?I}5z(c6NGS9(H1#yjhcIxTaXR`;5OfkuNEkflouv2`581ircw zRg=>(Qrxzv_Z@b;3Ic2j3UwVGmagSal`k_Ley>@c>h2k=S45aKd+7HMJS7z@&q4$Q zqzrZEU~Ty5mvy!;cma*Qld_+M9&yl2jEp#$Yy)Ngfh)piO{T-IrP7xs#-=x>6Qxe8WeXCm zx3MhOJzRFsD&}c7I|BMiWS#RL!;ikrDs_n3@dZZS|2^gw8o_a6a8F$=9rjsX|pndl#c0Z5& zoFJE%0jjt%5#0DGA?Ct<>-!)A*~IeTnlky)J&_ahjM#jjb7Ap&CyUPENa*h8rLC$a z9&d=U-O=}M9$8es3As){30ldLQlL{Ow;Oogf&wWaQ1MIWI&RawX0m%L=?u$Y;hbCn z$|-52{v1j|_}KTF+dvdh6gt!2vPm)eq$9=$G0+(;QYu&MAn4cdnm7ZfwpB6iT6}6P z>?&_*DZ(?x!AopyLFK)_q#yFE2qLCf;W2lhzW8alTY$szdERJ^6-o>Ol>Mc>Y)~LA zq(aOSX_YbS@NBe@Qr+3;&x-0zJQ?U0fXRm}P1;ttdv;}ajt(8VF)R0J;1x0dvFkxb z0PlU!RJQ$(&!5w7wScYXk^Jt)XzcE1Hm#p%20_aC!Zq(|g~4(5=8IpQ$n8*%$UQuS zR;D;)@U$+sg*fiFL4kyc>`lYNZ#Fo-YE;sy+J7Ga9Y+EFKCp!t{>TGiGHa zgMGjoeHRapDkj4cwU~I~GL!S0*N4OaQ=4mNt!by}(@7xuS@?0*(3%ip8Pu06mb+}8 z>8|nZuTBJUeMV9$z!=Q#j;dT4ZdVB;U_R8RE!VQUJI?3_R)@^3nQ@6%OJRth{*c=!80 zN!02dB;oHI2vAHYy01-ptFw|;a(w7Ey^d#Yimow$FXjmfX9wweSUiTQ$T_|8*fZkg zM>I(QEq!LVvF$Ym1s4QQ7Jo38hwyj#Dd$T4!_0+jdPgfbr+dYOJxX&1!xkXQ~g>Cru(&~YAeD+5tfNAXhAV!xYw?nOV#XGlZ}iZlym0~ zQGYF~)7&ejd1NHW$dWKn~T6lXcTg{}wijDfabj@RLv0AHbX?P#0 zz(EbLiiMUay(&q`!i@mx@I(txkE~O82}6COweOVt(b-X$2lS#?BaC-hfF^xD461>D zMaRu6gdCVm8=`(0fD~3!0FoyQwV8T#05|0fM-Tm$krSP1A#zPhT4~wy&G^@z{$JMD zPw_ht^T8^lAu-4KyDkP3=J;vNoc-(fZd}|~KAks6CX+GITOWo7TiF|UXWOMP0fy6s zW>SG3b(&9^K)cK=t!O3#2QW2Un&Vfx)nXG7|o#r0ubkO0zQnO@jl2H9SQiT{7r3JvWj(>4~g z0muivFftge^7P!uX70nY?ZUsd*@9W{P_qN;ifDtT@3OR)EPU|28?a9l&T*m|s!8xH z6yRU)z*p%y?bwy~wwOUcqLH{SQrFb0L*g)z%Tp4Yf-vTU3u-%>t13(kJ zeF;gA<>l{g{qWY(2HuMSp3|PK3T3|ta#cd;-#7hR4+JgvYAU^OQ!r)<6ac;j$HkRj z{YJIDv~v~`?;jddVa~hDL4F?4QZfxDB{C(|_E*hKS_PBQrj??vjJfzA8pkEvBr1OLISg;(BQ zj0@$$z(^&($Jrx{3r)~uy#J(+Yloh7SowPt26Y5LlIVdQ59N@umkmbC!mH>06R^bq zb2Br@0D6pM2WR_!Xdf51$_^1_Sd1zYfjTW8zNJ1@mD1pOg~{% zASH3G)_w8n^&saf|Mt)GI5@ZIp3QG%8^AM&*RtQ86Yr*eRxwWBM*WG0BBuWKZuy_? z!(Wv3We1qLaQbo`z|^!PZ`FJgq-*pLYPSD%=2NCfUc|j)vC%7d7FhQ;rTmL^0P7_Q zL!&Gioxha(8Hqacndl`zs`JG>s81(>|M&NRjUNE`!i6H-=KnjT<4+FE6a)C5`)zmX zzf0yn`^3Wb2A1E;xNH4?z5%%bK+#J4ZOFraRTIA-YDN)Q{$$`Y`@akJZxjQtY5+tI zTH%o2LIc0=?zaaV1M^l!l448$z>fGAiVY+Kd5BxzmU@5R)4zNVkUmzf1D}Y)8ot;1WxcB&oAug#Y;- zu=f*8Lztrc*q?(j{+C!5n0k>*KX(6*qrd(ilm6c^=~s;glgs0u9&BMDw3og zC$tRYzd1t=5j{T=E`X=zy$oDtzkS0FgUEDQD#ZN{xzA=J8JrPOuLp2Dx1xRIjnR_n z@GVWp($n18Uen9@YI(_mmcCe+3YUIYkUEJf_B%wVjFJkp(LekC7cB>QK1oQ^>F3TU!CGo=P z&0>^s3U_R@oe8X5#9Olu>sb(sL{X^kdVV7KeG5K5G#H7#5}y8tkg{-+44+88%XRTs zu$?Ey&1QKP5cVoR+IBU!`k>RchzU#O|M+2n0Z4UA5!Cz~Fv3r}Ba4K_y7R z)=#m4I_D)cL`E%P>dUZ;+ga?X16n{|5zuGj1J#&rG+$weD3TwhCI$B1(>$)Zu}n@9E76S(GLvpUS`N_q$_@ljK1L`^jVgh^;vkPTeGZO zPCGqUmS?B)>x{FS_dl*U(FHswNjtGIxz=$m2}%MDje6`2LUwHTTnx@Ez{&J`2kD=< zc_hz@lN#VK+L88Dp6%2ZrM-tsbU8Qb*m_#}B;P=slHa$N+09=3{~ODj&fF_f`_m;4 z#z&|85v)|>NA*z?$TWjJs*r^a@I0#!u&&|sL~YjH_0#{%wf9c^+L^POKYJbsMP zC9$1(1)I2j&C~m~WZOYYjw;k9M#yGh$YrrDt3(M;hDK%W2WQR)9DESz=CH%yZ@wr` znaoYT4<`G$ZX~^0xwY1R!iy`6N&YXj%%WT3Z!b4Q(XDU=G^~G1I@Ja(W~Vi*Ht^>9 zc5ZT$ZSj(V`%HzRZ?>=7D7=EnZ0gnwo_tF7y8$baHkB*SBNlp}|($ zjkleMn%nboZtk~;{Hd0xBOD43*nwH4&S44aL@MyGOW+CR%Q}^__ge7f1>ANkTy=Kh zUe_ZNg?BZtJ9HvvBkvKp`!R1gbyb~m%ceff_Rmmdsb1Z1QWqs!4kQ8xP3(AV@=&Lh z1UTY>H^#>mu5#F*oW9$lxLo#DS(i5O9Nwsk?H#8jqb$C$barH~SeBjlV4G=(zFBN^{a0UIKH+kh082o@u>@sCZh`OiYHJJ(l64AMUk7Sj zm|n)&zZlRx}Sw!NNh|9$(+IK#^QWg&8}YxmDz|2_VHP0!(9 zrRLA}NWb3yFfOk6-v7V<3Zx$YEBt%4?(Ku!^Zk?8%s*`XyyvtO@aRB82;5; zn;RmDAzTAed8Jjy1x;i*G4oq~FLX9U3TbVXUBgYtW7(&0M3k7A3Q{H>!*5vw!y7gXC!*LaASlqF+-<^!H{MxT63ij# z5|?3(9SsaJkxLA5$wQ1+VAze{9u5~4mIo)4dV!@BK81`ocoFe}Ed)lB4oW1W#T|{3 zj)6{nI0FM;xu=U`2=?e3O-6Xq^=Ju%FU^i7ozbL&5dx!?7rwU9XuAik-a~Gjj3%AY zq=T*H*f82;M$RVaIcT)Eg}r494ztmuGn#ZTLx4uTE$r!V#5R)~zopr0DT**%>V!Z